lucasbru commented on code in PR #20907:
URL: https://github.com/apache/kafka/pull/20907#discussion_r2585097590
##########
group-coordinator/src/main/java/org/apache/kafka/coordinator/group/streams/StreamsGroup.java:
##########
@@ -1027,12 +1027,28 @@ private void addTaskProcessIdFromActiveTasksWithEpochs(
if (partitionsOrNull == null) {
partitionsOrNull = new TimelineHashMap<>(snapshotRegistry,
assignedTaskPartitionsWithEpochs.size());
}
- for (Integer partitionId :
assignedTaskPartitionsWithEpochs.keySet()) {
- String prevValue = partitionsOrNull.put(partitionId,
processId);
- if (prevValue != null) {
- throw new IllegalStateException(
- String.format("Cannot set the process ID of %s-%s
to %s because the partition is " +
- "still owned by process ID %s", subtopologyId,
partitionId, processId, prevValue));
+ for (Map.Entry<Integer, Integer> partitionEntry :
assignedTaskPartitionsWithEpochs.entrySet()) {
+ Integer partitionId = partitionEntry.getKey();
+ String prevValue = partitionsOrNull.get(partitionId);
+
+ if (prevValue == null) {
+ partitionsOrNull.put(partitionId, processId);
+ } else {
+ String memberId = null;
+ for (Map.Entry<String, StreamsGroupMember> memberEntry
: members.entrySet()) {
+ if
(memberEntry.getValue().processId().equals(prevValue)) {
+ memberId = memberEntry.getKey();
+ break;
+ }
+ }
+ if (memberId != null &&
+
members.get(memberId).assignedTasks().activeTasksWithEpochs().get(subtopologyId).get(partitionId)
<= partitionEntry.getValue()) {
+ partitionsOrNull.put(partitionId, processId);
+ } else {
+ throw new IllegalStateException(
+ String.format("[GroupId {}] Cannot set the
process ID of {}-{} to {} because the partition is " +
+ "still owned by process ID {}", groupId,
subtopologyId, partitionId, processId, prevValue));
+ }
Review Comment:
Yes, we are using processIds here to guide the assignment (no stateful tasks
can be assigned to the same processID), but we don't identify a single member.
That could be added, of course, but we saw that this wasn't really used in
consumer groups. So I would support not doing this check here.
--
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: [email protected]
For queries about this service, please contact Infrastructure at:
[email protected]