KAFKA-10085: correctly compute lag for optimized source changelogs#8787
Conversation
|
Ok to test |
vvcephei
left a comment
There was a problem hiding this comment.
Thanks, @ableegoldman ! It LGTM overall. I had a couple of comments. Also, it seems like we're missing a fair amount of testing.
There was a problem hiding this comment.
This seems to be a step backwards, actually. Why wrap it as a StreamsException only just to immediately unwrap it again?
There was a problem hiding this comment.
I thought this might raise some eyebrows. I wanted to keep the ClientUtils methods consistent, and thought wrapping everything as a StreamsException would be cleaner. But maybe it makes more sense to throw the TimeoutException separately...
|
Hey @vvcephei, I addressed your comments and added tests. Let me know if there's any test coverage that still seems missing |
bc50399 to
859c4c5
Compare
There was a problem hiding this comment.
@vvcephei I've been wondering if maybe we should only catch the TimeoutException, and interpret a StreamsException as fatal (like IllegalStateException for example). This is how we were using Consumer#committed in the StoreChangelogReader, and AFAICT that only throws KafkaException on "unrecoverable errors" (quoted from javadocs)
But I can't tell whether the Admin's listOffsets might throw on transient errors, so I'm leaning towards catching both just to be safe. WDYT?
There was a problem hiding this comment.
That sounds reasonable, but I think if you throw an exception in the assignor, it just calls the assignor again in a tight loop, which seems worse than backing off and trying again later.
If you want to propose this change, maybe you can verify what exactly happens if we throw.
There was a problem hiding this comment.
if you throw an exception in the assignor, it just calls the assignor again in a tight loop
Wouldn't the leader thread just die? Not saying that that's ideal, either. But it's at least in line with how exceptions thrown by other admin client requests in the assignment are currently handled.
|
Test this please |
|
Java14 build passed, Java 11 and 8 builds failed with env issue |
859c4c5 to
2fa5d78
Compare
|
Test this please |
1 similar comment
|
Test this please |
vvcephei
left a comment
There was a problem hiding this comment.
Still haven’t managed to finish a review, but I did have this comment:
There was a problem hiding this comment.
Upon retrospect, I'm not sure if this is possible. The javadoc for Future#get indicates that any exception would be wrapped in an ExecutionException.
There was a problem hiding this comment.
Good catch. Do you think it should still be thrown/treated separately, though? See also my comment in StreamsPartitionAssignor below
|
Test this please |
0fb2736 to
303a6e7
Compare
|
Retest this please. |
1 similar comment
|
Retest this please. |
|
Retest this please. |
…8787) Split out the optimized source changelogs and fetch the committed offsets rather than the end offset for task lag computation Reviewers: John Roesler <vvcephei@apache.org>
* 'trunk' of github.com:apache/kafka: (42 commits) HOTFIX: Fix compile error in TopicAdminTest (apache#8866) KAFKA-10144: clean up corrupted standby tasks before attempting a commit (apache#8849) KAFKA-10157: Fix broken tests due to InterruptedException from FinalizedFeatureChangeListener (apache#8857) KAFKA-9432: automated protocol for DescribeConfigs (apache#8312) KAFKA-10049: Fixed FKJ bug where wrapped serdes are set incorrectly when using default StreamsConfig serdes (apache#8764) KAFKA-10027: Implement read path for feature versioning system (KIP-584) (apache#8680) KAFKA-10085: correctly compute lag for optimized source changelogs (apache#8787) KAFKA-10086: Integration test for ensuring warmups are effective (apache#8818) KAFKA-9374: Make connector interactions asynchronous (apache#8069) MINOR: reduce sizeInBytes for percentiles metrics (apache#8835) KAFKA-10115: Incorporate errors.tolerance with the Errant Record Reporter (apache#8829) KAFKA-9216: Enforce that Connect’s internal topics use `compact` cleanup policy (apache#8828) KAFKA-9845: Warn users about using config providers with plugin.path property (apache#8455) KAFKA-7833: Add missing test (apache#8847) KAFKA-9066: Retain metrics for failed tasks (apache#8502) KAFKA-9841: Revoke duplicate connectors and tasks when zombie workers return with an outdated assignment (apache#8453) KAFKA-9985: Sink connector may exhaust broker when writing in DLQ (apache#8663) KAFKA-9441: remove prepareClose() to simplify task management (apache#8833) KAFKA-7833: Add Global/StateStore name conflict check (apache#8825) KAFKA-9969: Exclude ConnectorClientConfigRequest from class loading isolation (apache#8630) ...
…t-for-generated-requests * apache-github/trunk: (248 commits) KAFKA-10049: Fixed FKJ bug where wrapped serdes are set incorrectly when using default StreamsConfig serdes (apache#8764) KAFKA-10027: Implement read path for feature versioning system (KIP-584) (apache#8680) KAFKA-10085: correctly compute lag for optimized source changelogs (apache#8787) KAFKA-10086: Integration test for ensuring warmups are effective (apache#8818) KAFKA-9374: Make connector interactions asynchronous (apache#8069) MINOR: reduce sizeInBytes for percentiles metrics (apache#8835) KAFKA-10115: Incorporate errors.tolerance with the Errant Record Reporter (apache#8829) KAFKA-9216: Enforce that Connect’s internal topics use `compact` cleanup policy (apache#8828) KAFKA-9845: Warn users about using config providers with plugin.path property (apache#8455) KAFKA-7833: Add missing test (apache#8847) KAFKA-9066: Retain metrics for failed tasks (apache#8502) KAFKA-9841: Revoke duplicate connectors and tasks when zombie workers return with an outdated assignment (apache#8453) KAFKA-9985: Sink connector may exhaust broker when writing in DLQ (apache#8663) KAFKA-9441: remove prepareClose() to simplify task management (apache#8833) KAFKA-7833: Add Global/StateStore name conflict check (apache#8825) KAFKA-9969: Exclude ConnectorClientConfigRequest from class loading isolation (apache#8630) KAFKA-9991: Fix flaky unit tests (apache#8843) KAFKA-10014; Always try to close all channels in Selector#close (apache#8685) KAFKA-10079: improve thread-level stickiness (apache#8775) MINOR: Print all removed dynamic members during join complete (apache#8816) ...
Split out the optimized source changelogs and fetch the committed offsets rather than the end offset for task lag computation
Must be cherrypicked to 2.6