[ https://issues.apache.org/jira/browse/KAFKA-7877?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=16756507#comment-16756507 ]
Arjun Satish commented on KAFKA-7877: ------------------------------------- This is not a bug. We decided to not implement DLQ to capture failures in the sink task for the following reasons: # When SinkTask#put() is invoked, a collection of SinkRecords are passed into it. It is not clear which of these records, if any, are the cause of the failure. In the case of a connection error, there are no bad records per se. Either the connector needs to be resilient to such connections and retry internally or throw a RetriableException. # RetriableExceptions thrown from the SinkTask are infinitely retried (with the same set of SinkRecords). Records must make it to the DLQ only when the framework has decided if it hits a hard error, and just cannot proceed. # In some cases, the records provided to SinkTask#put() could be cached by a Connector optimized for high throughput. A later call to this method causes an error, and it might have been due to the cached records. Maybe, we should add a new Exception type (say, BadDataException) that can return the set of bad records to the framework for more precise error reporting. Could you please point to the phrasing in the KIP that led to the confusion? I'll update it so the behavior is clearer. > Connect DLQ not used in SinkTask put() > -------------------------------------- > > Key: KAFKA-7877 > URL: https://issues.apache.org/jira/browse/KAFKA-7877 > Project: Kafka > Issue Type: Bug > Components: KafkaConnect > Affects Versions: 2.0.0, 2.0.1, 2.1.0 > Reporter: Andrew Bourgeois > Priority: Major > > The Dead Letter Queue isn't implemented for the put() operation. > In > [https://github.com/apache/kafka/blob/trunk/connect/runtime/src/main/java/org/apache/kafka/connect/runtime/WorkerSinkTask.java] > the "retryWithToleranceOperator" gets used during the conversion and > transformation phases, but not when delivering the messages to the sink task. > According to KIP-298, the Dead Letter Queue should be used as long as we > throw org.apache.kafka.connect.errors.RetriableException. > -- This message was sent by Atlassian JIRA (v7.6.3#76005)