[
https://issues.apache.org/jira/browse/FLINK-24530?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
]
Fabian Paul updated FLINK-24530:
--------------------------------
Description:
If a pipeline has a unified Sink that implements a {{Committer}} and
{{GlobalCommitter}} committables are forwarded from the {{Committer}} on either
receiving {{notifyCheckpointCompleted}} or {{endInput}}.
Since both methods are triggered by RPC calls we currently do not guarantee an
order when either method is called on the {{Committer}} or {{GlobalCommitter}}.
This can lead to that the {{GlobalCommitter}} receives the RPC call at first
and tries to commit although the {{Committer}} has nothing forwarded yet. For
the normal streaming execution, it means that the visibility of the final
commit could be delayed by one checkpoint.
Unfortunately, if the pipeline is stopped with a savepoint and drained the
{{endInput}} might be called at the {{GlobalCommitter}} before the
{{Committer}} leading to potential data loss.
was:
If a pipeline has a unified Sink that implements a {{Committer}} and
{{GlobalCommitter}} committables are forwarded from the {{Committer}} on either
receiving {{notifyCheckpointCompleted}} or {{endInput}}.
Since both methods are triggered by RPC calls we currently do not guarantee an
order when either method is called on the {{Committer}} or {{GlobalCommitter}}.
This can lead to that the {{GlobalCommitter}} receives the RPC call at first
and tries to commit although the {{Committer}} has nothing forwarded yet. For
the normal streaming execution, it means that the visibility of the final
commit could be delayed by one checkpoint.
Unfortunately, if the pipeline is stopped with a savepoint and drained the
{{endInput}} might be called at the {{GlobalCommitter}} before the
{{Committer}} leading to potential data loss.
With FLIP-147 (checkpoints after task finish) fully implemented the problem is
mitigated because it guarantees the order of the final
{{notifyCheckpointCompleted}} call although it still suffers from the
potentially delayed visibility.
> GlobalCommitter might not commit all records on drain
> -----------------------------------------------------
>
> Key: FLINK-24530
> URL: https://issues.apache.org/jira/browse/FLINK-24530
> Project: Flink
> Issue Type: Bug
> Components: API / DataStream, Connectors / Common
> Affects Versions: 1.14.0, 1.13.2, 1.15.0
> Reporter: Fabian Paul
> Priority: Major
>
> If a pipeline has a unified Sink that implements a {{Committer}} and
> {{GlobalCommitter}} committables are forwarded from the {{Committer}} on
> either receiving {{notifyCheckpointCompleted}} or {{endInput}}.
> Since both methods are triggered by RPC calls we currently do not guarantee
> an order when either method is called on the {{Committer}} or
> {{GlobalCommitter}}. This can lead to that the {{GlobalCommitter}} receives
> the RPC call at first and tries to commit although the {{Committer}} has
> nothing forwarded yet. For the normal streaming execution, it means that the
> visibility of the final commit could be delayed by one checkpoint.
> Unfortunately, if the pipeline is stopped with a savepoint and drained the
> {{endInput}} might be called at the {{GlobalCommitter}} before the
> {{Committer}} leading to potential data loss.
--
This message was sent by Atlassian Jira
(v8.3.4#803005)