[ https://issues.apache.org/jira/browse/FLINK-32362?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=17733363#comment-17733363 ]
Rui Fan commented on FLINK-32362: --------------------------------- Thanks for your quick feedback. I'm not sure what should we do when some subtasks are not ready. * Option1: just send event to all ready subtasks. * Option2: Don't send any event before all subtasks are ready. If we expect option1, your solution might work. And your solution1 has a little bug, when one subtask isn't ready, we should continue send event to next subtask. So the `try catch` should be moved to inside of the `for loop`. {code:java} for (Integer subtaskId : subTaskIds) { try { context.sendEventToSourceOperator( subtaskId, new WatermarkAlignmentEvent(maxAllowedWatermark)); } catch (Throwable ignore) { LOG.warn("Announce the newest combined watermark to source failed, task maybe during failover, wait next time to announce." } } {code} > SourceAlignment announceCombinedWatermark period task maybe lost > ---------------------------------------------------------------- > > Key: FLINK-32362 > URL: https://issues.apache.org/jira/browse/FLINK-32362 > Project: Flink > Issue Type: Bug > Affects Versions: 1.16.0 > Reporter: Cai Liuyang > Assignee: Cai Liuyang > Priority: Major > > When we use sourcealignment,we also found there is another problem that > announceCombinedWatermark may throw a exception (like "subtask 25 is not > ready yet to receive events" , this subtask maybe under failover), which will > lead the period task not running any more (ThreadPoolExecutor will not > schedule the period task if it throw a exception) > I think we should increase the robustness of announceCombinedWatermark > function to avoid it throw any exception (if send fail, just wait next send) > (code see > [https://github.com/apache/flink/blob/master/flink-runtime/src/main/java/org/apache/flink/runtime/source/coordinator/SourceCoordinator.java#L199] > ) -- This message was sent by Atlassian Jira (v8.20.10#820010)