Class SinkRecord


public class SinkRecord extends ConnectRecord<SinkRecord>
SinkRecord is a ConnectRecord that has been read from Kafka and includes the original Kafka record's topic, partition and offset (before any transformations have been applied) in addition to the standard fields. This information should be used by the SinkTask to coordinate offset commits.

It also includes the TimestampType, which may be TimestampType.NO_TIMESTAMP_TYPE, and the relevant timestamp, which may be null.

  • Constructor Details

    • SinkRecord

      public SinkRecord(String topic, int partition, Schema keySchema, Object key, Schema valueSchema, Object value, long kafkaOffset)
    • SinkRecord

      public SinkRecord(String topic, int partition, Schema keySchema, Object key, Schema valueSchema, Object value, long kafkaOffset, Long timestamp, org.apache.kafka.common.record.TimestampType timestampType)
    • SinkRecord

      public SinkRecord(String topic, int partition, Schema keySchema, Object key, Schema valueSchema, Object value, long kafkaOffset, Long timestamp, org.apache.kafka.common.record.TimestampType timestampType, Iterable<Header> headers)
    • SinkRecord

      public SinkRecord(String topic, int partition, Schema keySchema, Object key, Schema valueSchema, Object value, long kafkaOffset, Long timestamp, org.apache.kafka.common.record.TimestampType timestampType, Iterable<Header> headers, String originalTopic, Integer originalKafkaPartition, long originalKafkaOffset)
      This constructor is intended for use by the Connect runtime only and plugins (sink connectors or transformations) should not use this directly outside testing code.
  • Method Details

    • kafkaOffset

      public long kafkaOffset()
    • timestampType

      public org.apache.kafka.common.record.TimestampType timestampType()
    • originalTopic

      public String originalTopic()
      Get the original topic for this sink record, before any transformations were applied. In order to be compatible with transformations that mutate topic names, this method should be used by sink tasks instead of ConnectRecord.topic() for any internal offset tracking purposes (for instance, reporting offsets to the Connect runtime via SinkTask.preCommit(Map)).

      This method was added in Apache Kafka 3.6. Sink connectors that use this method but want to maintain backward compatibility in order to be able to be deployed on older Connect runtimes should guard the call to this method with a try-catch block, since calling this method will result in a NoSuchMethodError when the sink connector is deployed to Connect runtimes older than Kafka 3.6. For example:

      
       String originalTopic;
       try {
           originalTopic = record.originalTopic();
       } catch (NoSuchMethodError e) {
           log.warn("This connector is not compatible with SMTs that mutate topic names, topic partitions or offset values on this version of Kafka Connect");
           originalTopic = record.topic();
       }
       
       

      Note that sink connectors that do their own offset tracking will be incompatible with SMTs that mutate topic names when deployed to older Connect runtimes that do not support this method.

      Returns:
      the topic for this record before any transformations were applied
      Since:
      3.6
    • originalKafkaPartition

      public Integer originalKafkaPartition()
      Get the original topic partition for this sink record, before any transformations were applied. In order to be compatible with transformations that mutate topic partitions, this method should be used by sink tasks instead of ConnectRecord.kafkaPartition() for any internal offset tracking purposes (for instance, reporting offsets to the Connect runtime via SinkTask.preCommit(Map)).

      This method was added in Apache Kafka 3.6. Sink connectors that use this method but want to maintain backward compatibility in order to be able to be deployed on older Connect runtimes should guard the call to this method with a try-catch block, since calling this method will result in a NoSuchMethodError when the sink connector is deployed to Connect runtimes older than Kafka 3.6. For example:

      
       String originalKafkaPartition;
       try {
           originalKafkaPartition = record.originalKafkaPartition();
       } catch (NoSuchMethodError e) {
           log.warn("This connector is not compatible with SMTs that mutate topic names, topic partitions or offset values on this version of Kafka Connect");
           originalKafkaPartition = record.kafkaPartition();
       }
       
       

      Note that sink connectors that do their own offset tracking will be incompatible with SMTs that mutate topic partitions when deployed to older Connect runtimes that do not support this method.

      Returns:
      the topic partition for this record before any transformations were applied
      Since:
      3.6
    • originalKafkaOffset

      public long originalKafkaOffset()
      Get the original offset for this sink record, before any transformations were applied. In order to be compatible with transformations that mutate offset values, this method should be used by sink tasks instead of kafkaOffset() for any internal offset tracking purposes (for instance, reporting offsets to the Connect runtime via SinkTask.preCommit(Map)).

      This method was added in Apache Kafka 3.6. Sink connectors that use this method but want to maintain backward compatibility in order to be able to be deployed on older Connect runtimes should guard the call to this method with a try-catch block, since calling this method will result in a NoSuchMethodError when the sink connector is deployed to Connect runtimes older than Kafka 3.6. For example:

      
       String originalKafkaOffset;
       try {
           originalKafkaOffset = record.originalKafkaOffset();
       } catch (NoSuchMethodError e) {
           log.warn("This connector is not compatible with SMTs that mutate topic names, topic partitions or offset values on this version of Kafka Connect");
           originalKafkaOffset = record.kafkaOffset();
       }
       
       

      Note that sink connectors that do their own offset tracking will be incompatible with SMTs that mutate offset values when deployed to older Connect runtimes that do not support this method.

      Returns:
      the offset for this record before any transformations were applied
      Since:
      3.6
    • newRecord

      public SinkRecord newRecord(String topic, Integer kafkaPartition, Schema keySchema, Object key, Schema valueSchema, Object value, Long timestamp)
      Description copied from class: ConnectRecord
      Create a new record of the same type as itself, with the specified parameter values. All other fields in this record will be copied over to the new record. Since the headers are mutable, the resulting record will have a copy of this record's headers.
      Specified by:
      newRecord in class ConnectRecord<SinkRecord>
      Parameters:
      topic - the name of the topic; may be null
      kafkaPartition - the partition number for the Kafka topic; may be null
      keySchema - the schema for the key; may be null
      key - the key; may be null
      valueSchema - the schema for the value; may be null
      value - the value; may be null
      timestamp - the timestamp; may be null
      Returns:
      the new record
    • newRecord

      public SinkRecord newRecord(String topic, Integer kafkaPartition, Schema keySchema, Object key, Schema valueSchema, Object value, Long timestamp, Iterable<Header> headers)
      Description copied from class: ConnectRecord
      Create a new record of the same type as itself, with the specified parameter values. All other fields in this record will be copied over to the new record.
      Specified by:
      newRecord in class ConnectRecord<SinkRecord>
      Parameters:
      topic - the name of the topic; may be null
      kafkaPartition - the partition number for the Kafka topic; may be null
      keySchema - the schema for the key; may be null
      key - the key; may be null
      valueSchema - the schema for the value; may be null
      value - the value; may be null
      timestamp - the timestamp; may be null
      headers - the headers; may be null or empty
      Returns:
      the new record
    • equals

      public boolean equals(Object o)
      Overrides:
      equals in class ConnectRecord<SinkRecord>
    • hashCode

      public int hashCode()
      Overrides:
      hashCode in class ConnectRecord<SinkRecord>
    • toString

      public String toString()
      Overrides:
      toString in class ConnectRecord<SinkRecord>