Skip to content

Optimize MSQ realtime queries#15399

Merged
cryptoe merged 20 commits intoapache:masterfrom
adarshsanjeev:realtime-msq-optimization
Feb 28, 2024
Merged

Optimize MSQ realtime queries#15399
cryptoe merged 20 commits intoapache:masterfrom
adarshsanjeev:realtime-msq-optimization

Conversation

@adarshsanjeev
Copy link
Copy Markdown
Contributor

@adarshsanjeev adarshsanjeev commented Nov 20, 2023

Major changes

Currently, while reading results from realtime tasks, requests are sent on a segment level. This is slightly wasteful, as when contacting a data servers, it is possible to transfer results for all segments which it is hosting, instead of only one segment at a time.

One change this PR makes is to group the segments on the basis of servers. This reduces the number of queries to data servers made. Since we don't have access to the number of rows for realtime segments, the grouping is done with a fixed estimated number of rows for each realtime segment.

If the segments have already been handed off, the earlier approach made was to fetch the segments from deep storage.

After the above changes, a single worker processor thread might have to fetch all segments from deepstorage, missing an oppurtunity for parellelism, since each segment fetch from deepstorage, is a separate operation. This PR also edits the ChainedProcessorManager. The initial BaseLeafFrameProcessors return the list of segments which were handed off. The ProcessorManager then uses this information to create new list of frame processors, handle the newer handed off segments.

  • Optimize MSQ queries to realtime tasks, by grouping segments served by the same server together.
  • Modify BaseLeafFrameProcessors and ChainedProcessorManager to take handle handed of segments
  • Update DataServerResponseHandler to support backpressure and query timeout.

This PR has:

  • been self-reviewed.
  • added documentation for new or modified features or behaviors.
  • a release note entry in the PR description.
  • added Javadocs for most classes and all non-trivial methods. Linked related entities via Javadoc links.
  • added or updated version, license, or notice information in licenses.yaml
  • added comments explaining the "why" and the intent of the code wherever would not be obvious for an unfamiliar reader.
  • added unit tests or modified existing tests to cover new code paths, ensuring the threshold for code coverage is met.
  • added integration tests.
  • been tested in a test Druid cluster.

@github-actions github-actions Bot added Area - Batch Ingestion Area - MSQ For multi stage queries - https://github.com/apache/druid/issues/12262 labels Nov 20, 2023
Comment on lines +268 to +285
queue.offer(
InputStreamHolder.fromStream(
new InputStream()
{
@Override
public int read() throws IOException
{
if (th != null) {
throw new IOException(msg, th);
} else {
throw new IOException(msg);
}
}
},
-1,
0
)
);

Check notice

Code scanning / CodeQL

Ignored error status of call

Method setupResponseReadFailure ignores exceptional return value of BlockingQueue<InputStreamHolder>.offer.
Copy link
Copy Markdown
Contributor

@cryptoe cryptoe left a comment

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Overall LGTM. Left some comments.

// No cursors!
cursorYielder.close();
return ReturnOrAwait.returnObject(Unit.instance());
return ReturnOrAwait.returnObject(handedOffSegments);
Copy link
Copy Markdown
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

If there are not handed of segments nulls will be passed. Is that intended ?

@Override
public long getWeight()
{
return segments.size() * DATA_SERVER_WEIGHT_ESTIMATION;
Copy link
Copy Markdown
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

We are estimating the number of rows to be 5000 for realtime segments rite ?

Copy link
Copy Markdown
Contributor Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Yes, added a comment

}
List<InputSlice> handedOffSegments = new ArrayList<>();
for (Object o : objects) {
if (o instanceof SegmentsInputSlice) {
Copy link
Copy Markdown
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

O can be null as well.

Copy link
Copy Markdown
Contributor Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Added a null check

}

private void onFirstProcessorComplete(final Object firstResult)
private synchronized void checkFirstProcessorComplete()
Copy link
Copy Markdown
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Why is synchronized here ?

Copy link
Copy Markdown
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Please add documentation regarding which threads need this.

* Manager that chains processors: runs all processors generated by {@link #first} first, then based on its result,
* creates {@link #restFuture} using {@link #restFactory} and runs that next.
*/
public class ChainedProcessorManager<A, B, R> implements ProcessorManager<Object, R>
Copy link
Copy Markdown
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

@gianm Can you please vet the changes in this class.

@cryptoe cryptoe added this to the Druid 29.0.0 milestone Jan 17, 2024
@abhishekagarwal87 abhishekagarwal87 removed this from the 29.0.0 milestone Jan 30, 2024
@cryptoe cryptoe merged commit d2c2036 into apache:master Feb 28, 2024
@JsonProperty("dataSource") String dataSource,
@JsonProperty("segments") List<RichSegmentDescriptor> descriptors
@JsonProperty("segments") List<RichSegmentDescriptor> descriptors,
@JsonProperty("servedSegments") List<DataServerRequestDescriptor> servedSegments
Copy link
Copy Markdown
Contributor

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

This should be marked nullable for rolling upgrade cases.

@adarshsanjeev adarshsanjeev added this to the 30.0.0 milestone May 6, 2024
Sign up for free to join this conversation on GitHub. Already have an account? Sign in to comment

Labels

Area - Batch Ingestion Area - MSQ For multi stage queries - https://github.com/apache/druid/issues/12262 Area - Querying

Projects

None yet

Development

Successfully merging this pull request may close these issues.

4 participants