-
Notifications
You must be signed in to change notification settings - Fork 3.8k
Introduce published segment cache in broker #6901
New issue
Have a question about this project? Sign up for a free GitHub account to open an issue and contact its maintainers and the community.
By clicking “Sign up for GitHub”, you agree to our terms of service and privacy statement. We’ll occasionally send you account related emails.
Already on GitHub? Sign in to your account
Changes from 1 commit
e8b0a23
6ec4911
8b6d453
281600d
cdc751e
8fed80a
0d0f89f
8fc84b3
adf133f
d46edc5
92601d0
a993482
a4cbcfc
e376df3
0fbb48c
8df2d96
d5b7d79
2bfa396
632d741
bcc6513
c41f085
07a80b1
d83eb33
a440c0c
b385345
981b080
3a94cae
0032a31
ad28458
33751a4
2ddb7a1
1b37493
60dbf41
8860053
ca21779
3a70f39
e2a9af7
File filter
Filter by extension
Conversations
Jump to
Diff view
Diff view
There are no files selected for viewing
| Original file line number | Diff line number | Diff line change |
|---|---|---|
|
|
@@ -106,10 +106,15 @@ public void start() | |
| } | ||
| try { | ||
| if (isCacheEnabled) { | ||
| scheduledExec.schedule(new PollTask(), 0, TimeUnit.MILLISECONDS); | ||
| lifecycleLock.started(); | ||
| log.info("MetadataSegmentView Started."); | ||
| try { | ||
| poll(); | ||
| } | ||
| finally { | ||
|
||
| scheduledExec.schedule(new PollTask(), 0, TimeUnit.MILLISECONDS); | ||
|
||
| } | ||
| } | ||
| lifecycleLock.started(); | ||
| log.info("MetadataSegmentView Started."); | ||
| } | ||
| finally { | ||
| lifecycleLock.exitStart(); | ||
|
|
@@ -161,6 +166,10 @@ private void poll() | |
| public Iterator<DataSegment> getPublishedSegments() | ||
| { | ||
| if (isCacheEnabled) { | ||
| Preconditions.checkState( | ||
| lifecycleLock.awaitStarted(1, TimeUnit.MILLISECONDS), | ||
|
||
| "hold on, still syncing published segments" | ||
| ); | ||
| return publishedSegments.keySet().iterator(); | ||
| } else { | ||
| return getMetadataSegments( | ||
|
|
@@ -173,7 +182,7 @@ public Iterator<DataSegment> getPublishedSegments() | |
| } | ||
|
|
||
| // Note that coordinator must be up to get segments | ||
|
Member
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. See #7391
Author
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. will resolve with #7391 , thanks. |
||
| private static JsonParserIterator<DataSegment> getMetadataSegments( | ||
| private JsonParserIterator<DataSegment> getMetadataSegments( | ||
| DruidLeaderClient coordinatorClient, | ||
| ObjectMapper jsonMapper, | ||
| BytesAccumulatingResponseHandler responseHandler, | ||
|
|
@@ -226,20 +235,23 @@ private class PollTask implements Runnable | |
| @Override | ||
| public void run() | ||
| { | ||
| long delayMS = pollPeriodinMS; | ||
| try { | ||
| final long pollStartTime = System.nanoTime(); | ||
| poll(); | ||
| final long pollEndTime = System.nanoTime(); | ||
| final long pollTimeNS = pollEndTime - pollStartTime; | ||
| final long pollTimeMS = TimeUnit.NANOSECONDS.toMillis(pollTimeNS); | ||
| final long delayMS = Math.max(pollPeriodinMS - pollTimeMS, 0); | ||
| if (!Thread.currentThread().isInterrupted()) { | ||
| scheduledExec.schedule(new PollTask(), delayMS, TimeUnit.MILLISECONDS); | ||
| } | ||
| delayMS = Math.max(pollPeriodinMS - pollTimeMS, 0); | ||
| } | ||
| catch (Exception e) { | ||
| log.makeAlert(e, "Problem polling Coordinator.").emit(); | ||
| } | ||
| finally { | ||
| if (!Thread.currentThread().isInterrupted()) { | ||
| scheduledExec.schedule(new PollTask(), delayMS, TimeUnit.MILLISECONDS); | ||
|
Member
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. Why doesn't this class use
Author
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. i changed to use
Member
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. Both |
||
| } | ||
| } | ||
| } | ||
| } | ||
|
|
||
|
|
||
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
Looks like this class violates the Single Responsibility Principle, see #7392.
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
hmm I need to read up more on Single Responsibility Principle, another way to look at it, is the responsibility of this class is to get metadata segments, the public method is
getPublishedSegments, then it kind of does one thing i.e. get metadata segments.There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
Probably SRP is a wrong link in this case, cohesion may be a more correct reference. Anyway, it seems to me that
isCacheEnabledreplaces a perfectly healthy polymorphism "CachingMetadataSegmentView"/"DirectMetadataSegmentView" here. Somebody may want to add "CachingWithDirectMetadataStoreAccess" (vs. using Coordinator as a proxy), etc.