lianetm commented on code in PR #14690: URL: https://github.com/apache/kafka/pull/14690#discussion_r1383551733
########## clients/src/main/java/org/apache/kafka/clients/consumer/internals/MembershipManager.java: ########## @@ -73,30 +84,55 @@ public interface MembershipManager { /** * @return Current assignment for the member. */ - ConsumerGroupHeartbeatResponseData.Assignment currentAssignment(); + Set<TopicPartition> currentAssignment(); /** - * Update the assignment for the member, indicating that the provided assignment is the new - * current assignment. + * @return Assignment that the member received from the server but hasn't finished processing + * yet. */ - void onTargetAssignmentProcessComplete(ConsumerGroupHeartbeatResponseData.Assignment assignment); + Optional<ConsumerGroupHeartbeatResponseData.Assignment> targetAssignment(); /** - * Transition the member to the FENCED state and update the member info as required. This is - * only invoked when the heartbeat returns a FENCED_MEMBER_EPOCH or UNKNOWN_MEMBER_ID error. - * code. + * Transition to the {@link MemberState#JOINING} state, indicating that the member will + * try to join the group on the next heartbeat request. This is expected to be invoked when + * the user calls the subscribe API, or when the member wants to rejoin after getting fenced. + */ + void transitionToJoinGroup(); + + /** + * Transition the member to the FENCED state, where the member will release the assignment by + * calling the onPartitionsLost callback, and when the callback completes, it will transition + * to {@link MemberState#JOINING} to rejoin the group. This is expected to be invoked when + * the heartbeat returns a FENCED_MEMBER_EPOCH or UNKNOWN_MEMBER_ID error. */ void transitionToFenced(); /** * Transition the member to the FAILED state and update the member info as required. This is * invoked when un-recoverable errors occur (ex. when the heartbeat returns a non-retriable - * error or when errors occur while executing the user-provided callbacks) + * error) */ void transitionToFailed(); /** - * @return True if the member should send heartbeat to the coordinator. + * Release assignment and transition to {@link MemberState#LEAVING_GROUP} so that a heartbeat + * request is sent indicating the broker that the member wants to leave the group. This is + * expected to be invoked when the user calls the unsubscribe API. + * + * @return Future that will complete when the callback execution completes and the heartbeat + * to leave the group has been sent out. + */ + CompletableFuture<Void> leaveGroup(); Review Comment: Just for the record, it wasn't integrated initially when you took a look but it was added in this PR. And yes, you're right, it is integrated via the ApplicationEventProcessor and the HBManager -- This is an automated message from the Apache Git Service. To respond to the message, please log on to GitHub and use the URL above to go to the specific comment. To unsubscribe, e-mail: jira-unsubscr...@kafka.apache.org For queries about this service, please contact Infrastructure at: us...@infra.apache.org