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
MINOR: A few cleanups for DescribeQuorum APIs #12548
Conversation
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.
Thanks for the changes. Wanted to give you a quick review. Working on a more thorough review.
clients/src/main/resources/common/message/DescribeQuorumResponse.json
Outdated
Show resolved
Hide resolved
clients/src/main/resources/common/message/DescribeQuorumResponse.json
Outdated
Show resolved
Hide resolved
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.
Thanks for the PR. Left some comments.
private final Integer leaderEpoch; | ||
private final Long highWatermark; | ||
private final int leaderId; | ||
private final long leaderEpoch; |
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.
The leaderEpoch
is in int32 type, any reason we change to use long
here?
kafka/clients/src/main/resources/common/message/DescribeQuorumResponse.json
Lines 37 to 38 in add4ca6
{ "name": "LeaderEpoch", "type": "int32", "versions": "0+", | |
"about": "The latest known leader epoch"}, |
return leaderId.equals(that.leaderId) | ||
&& voters.equals(that.voters) | ||
&& observers.equals(that.observers); | ||
return leaderId == that.leaderId | ||
&& leaderEpoch == that.leaderEpoch | ||
&& highWatermark == that.highWatermark | ||
&& Objects.equals(highWatermarkUpdateTimeMs, that.highWatermarkUpdateTimeMs) | ||
&& Objects.equals(voters, that.voters) | ||
&& Objects.equals(observers, that.observers); |
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.
nice catch!
|LeaderId: ${quorumInfo.leaderId} | ||
|LeaderEpoch: ${quorumInfo.leaderEpoch} | ||
|HighWatermark: ${quorumInfo.highWatermark} | ||
|HighWatermarkUpdateTimeMs: ${quorumInfo.highWatermarkUpdateTimeMs} |
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.
Since this is a script output, I think it'd better we did some pre-processing to avoid printing out something like: OptionalLong[12345]
or OptionalLong.empty
.
@@ -304,7 +311,7 @@ public long epochStartOffset() { | |||
return epochStartOffset; | |||
} | |||
|
|||
private ReplicaState getReplicaState(int remoteNodeId) { | |||
private ReplicaState getOrCreateReplicaState(int remoteNodeId) { |
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.
nice renaming
DescribeQuorumResponseData.ReplicaState leaderState = voterStates.stream() | ||
.filter(voterState -> voterState.replicaId() == localId) | ||
.findFirst() | ||
.orElseThrow(() -> new AssertionError("")); |
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.
Forgot to add the error message here and below.
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.
Please update it and below. Thanks.
Also, some |
This is a very simple problem, we changed the output of the tool so the assertion in the test case failed, we only need to adjust the test case. |
Just a general note, I'm repurposing this patch as a general cleanup of the new DescribeQuorum APIs. I changed my mind about |
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.
Thanks for the changes @hachikuji . I still need to review the tests.
clients/src/main/java/org/apache/kafka/clients/admin/QuorumInfo.java
Outdated
Show resolved
Hide resolved
clients/src/main/java/org/apache/kafka/clients/admin/QuorumInfo.java
Outdated
Show resolved
Hide resolved
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.
LGTM thanks for the improvement @hachikuji
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.
@hachikuji , thanks for explaining why leaderEpoch
change to long
. But there are 2 comments you didn't address, yet. Please take a look. Thanks.
DescribeQuorumResponseData.ReplicaState leaderState = voterStates.stream() | ||
.filter(voterState -> voterState.replicaId() == localId) | ||
.findFirst() | ||
.orElseThrow(() -> new AssertionError("")); |
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.
Please update it and below. Thanks.
@showuon Apologies for missing your comments. I pushed an update to address them. Thanks for reviewing! |
A few small cleanups in the `DescribeQuorum` API and handling logic: - Change field types in `QuorumInfo`: - `leaderId`: `Integer` -> `int` - `leaderEpoch`: `Integer` -> `long` (to allow for type expansion in the future) - `highWatermark`: `Long` -> `long` - Use field names `lastFetchTimestamp` and `lastCaughtUpTimestamp` consistently - Move construction of `DescribeQuorumResponseData.PartitionData` into `LeaderState` - Consolidate fetch time/offset update logic into `LeaderState.ReplicaState.updateFollowerState` Reviewers: Luke Chen <showuon@gmail.com>, José Armando García Sancio <jsancio@users.noreply.github.com>
Thanks @hachikuji , LGTM! Nice refactor! |
A few small cleanups:
QuorumInfo
:leaderId
:Integer
->int
leaderEpoch
:Integer
->long
(to allow for type expansion in the future)highWatermark
:Long
->long
lastFetchTimestamp
andlastCaughtUpTimestamp
consistentlyDescribeQuorumResponseData.PartitionData
intoLeaderState
LeaderState.ReplicaState.updateFollowerState
Committer Checklist (excluded from commit message)