urbandan commented on a change in pull request #11748:
URL: https://github.com/apache/kafka/pull/11748#discussion_r815676139



##########
File path: 
connect/mirror/src/main/java/org/apache/kafka/connect/mirror/MirrorCheckpointTask.java
##########
@@ -169,6 +172,7 @@ public String version() {
         return listConsumerGroupOffsets(group).entrySet().stream()
             .filter(x -> shouldCheckpointTopic(x.getKey().topic()))
             .map(x -> checkpoint(group, x.getKey(), x.getValue()))
+            .flatMap(o -> o.map(Stream::of).orElseGet(Stream::empty)) // do 
not emit checkpoints for partitions that don't have offset-syncs

Review comment:
       There is a "feature" in checkpointing which copies the offsets of 
replica topics back to target as-is. (I'm using quotes because the copy doesn't 
really make any sense, the offsets should be properly translated.) E.g. there 
are clusters A and B, replication enabled in both direction. Topic "test" is 
replicated A->B, which creates "A.topic" in B. If the topic filter matches 
"A.topic", the topic itself won't be replicated, but checkpointing can still 
pick it up (there is no cycle detection in the shouldCheckpointTopic method). 
In its current state, checkpointing will handle this as a special case, and 
will copy the committed offset of "A.topic" to A, but without the prefix (there 
is also a bug in the implementation, which strips all prefixes instead of the 
last one).
   I think this feature is not useful at all (and also has a bug). My question 
is, whether that feature will be dropped, or somehow handles in this change as 
well. "A.topic" does not have an offset mapping in the offset-syncs topic, as 
it is never replicated back to A - this flatMap will filter all such topics.
   I think this should be handled separately, or the feature should be removed 
completely - in that case the MirrorCheckpointTask.renameTopicPartition method 
should be updated accordingly.




-- 
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


Reply via email to