[ https://issues.apache.org/jira/browse/FLINK-4027?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=15344621#comment-15344621 ]
ASF GitHub Bot commented on FLINK-4027: --------------------------------------- Github user rmetzger commented on a diff in the pull request: https://github.com/apache/flink/pull/2108#discussion_r68082639 --- Diff: flink-streaming-connectors/flink-connector-kafka-base/src/main/java/org/apache/flink/streaming/connectors/kafka/FlinkKafkaProducerBase.java --- @@ -50,11 +53,13 @@ /** * Flink Sink to produce data into a Kafka topic. * - * Please note that this producer does not have any reliability guarantees. + * Please note that this producer provides at-least-once reliability guarantees when + * checkpoints are enabled and setFlushOnCheckpoint(true) is set. + * Otherwise, the producer doesn't provide any reliability guarantees. --- End diff -- My reasoning here was that we first provide this as an optional feature to those users who know what they are doing / what they need to give the feature exposure. I want to be certain that it works in all environments before we activate it by default. > FlinkKafkaProducer09 sink can lose messages > ------------------------------------------- > > Key: FLINK-4027 > URL: https://issues.apache.org/jira/browse/FLINK-4027 > Project: Flink > Issue Type: Bug > Components: Kafka Connector > Affects Versions: 1.0.3 > Reporter: Elias Levy > Assignee: Robert Metzger > Priority: Critical > > The FlinkKafkaProducer09 sink appears to not offer at-least-once guarantees. > The producer is publishing messages asynchronously. A callback can record > publishing errors, which will be raised when detected. But as far as I can > tell, there is no barrier to wait for async errors from the sink when > checkpointing or to track the event time of acked messages to inform the > checkpointing process. > If a checkpoint occurs while there are pending publish requests, and the > requests return a failure after the checkpoint occurred, those message will > be lost as the checkpoint will consider them processed by the sink. -- This message was sent by Atlassian JIRA (v6.3.4#6332)