IN
- type of the records written to Kafka@PublicEvolving public class KafkaSink<IN> extends Object implements StatefulSink<IN,org.apache.flink.connector.kafka.sink.KafkaWriterState>, TwoPhaseCommittingSink<IN,org.apache.flink.connector.kafka.sink.KafkaCommittable>
DeliveryGuarantee
.
DeliveryGuarantee.NONE
does not provide any guarantees: messages may be lost in case
of issues on the Kafka broker and messages may be duplicated in case of a Flink failure.
DeliveryGuarantee.AT_LEAST_ONCE
the sink will wait for all outstanding records in the
Kafka buffers to be acknowledged by the Kafka producer on a checkpoint. No messages will be
lost in case of any issue with the Kafka brokers but messages may be duplicated when Flink
restarts.
DeliveryGuarantee.EXACTLY_ONCE
: In this mode the KafkaSink will write all messages in
a Kafka transaction that will be committed to Kafka on a checkpoint. Thus, if the consumer
reads only committed data (see Kafka consumer config isolation.level), no duplicates will be
seen in case of a Flink restart. However, this delays record writing effectively until a
checkpoint is written, so adjust the checkpoint duration accordingly. Please ensure that you
use unique transactionalIdPrefix
s across your applications running on the same Kafka
cluster such that multiple running jobs do not interfere in their transactions! Additionally,
it is highly recommended to tweak Kafka transaction timeout (link) >> maximum checkpoint
duration + maximum restart duration or data loss may happen when Kafka expires an uncommitted
transaction.on how to construct a KafkaSink
,
Serialized FormStatefulSink.StatefulSinkWriter<InputT,WriterStateT>, StatefulSink.WithCompatibleState
TwoPhaseCommittingSink.PrecommittingSinkWriter<InputT,CommT>
Sink.InitContext
Modifier and Type | Method and Description |
---|---|
static <IN> KafkaSinkBuilder<IN> |
builder()
Create a
KafkaSinkBuilder to construct a new KafkaSink . |
Committer<org.apache.flink.connector.kafka.sink.KafkaCommittable> |
createCommitter()
Creates a
Committer that permanently makes the previously written data visible
through Committer.commit(Collection) . |
org.apache.flink.connector.kafka.sink.KafkaWriter<IN> |
createWriter(Sink.InitContext context)
Create a
StatefulSink.StatefulSinkWriter . |
SimpleVersionedSerializer<org.apache.flink.connector.kafka.sink.KafkaCommittable> |
getCommittableSerializer()
Returns the serializer of the committable type.
|
protected Properties |
getKafkaProducerConfig() |
SimpleVersionedSerializer<org.apache.flink.connector.kafka.sink.KafkaWriterState> |
getWriterStateSerializer()
Any stateful sink needs to provide this state serializer and implement
StatefulSink.StatefulSinkWriter.snapshotState(long) properly. |
org.apache.flink.connector.kafka.sink.KafkaWriter<IN> |
restoreWriter(Sink.InitContext context,
Collection<org.apache.flink.connector.kafka.sink.KafkaWriterState> recoveredState)
Create a
StatefulSink.StatefulSinkWriter from a recovered state. |
public static <IN> KafkaSinkBuilder<IN> builder()
KafkaSinkBuilder
to construct a new KafkaSink
.IN
- type of incoming recordsKafkaSinkBuilder
@Internal public Committer<org.apache.flink.connector.kafka.sink.KafkaCommittable> createCommitter() throws IOException
TwoPhaseCommittingSink
Committer
that permanently makes the previously written data visible
through Committer.commit(Collection)
.createCommitter
in interface TwoPhaseCommittingSink<IN,org.apache.flink.connector.kafka.sink.KafkaCommittable>
IOException
- for any failure during creation.@Internal public SimpleVersionedSerializer<org.apache.flink.connector.kafka.sink.KafkaCommittable> getCommittableSerializer()
TwoPhaseCommittingSink
getCommittableSerializer
in interface TwoPhaseCommittingSink<IN,org.apache.flink.connector.kafka.sink.KafkaCommittable>
@Internal public org.apache.flink.connector.kafka.sink.KafkaWriter<IN> createWriter(Sink.InitContext context) throws IOException
StatefulSink
StatefulSink.StatefulSinkWriter
.createWriter
in interface Sink<IN>
createWriter
in interface StatefulSink<IN,org.apache.flink.connector.kafka.sink.KafkaWriterState>
createWriter
in interface TwoPhaseCommittingSink<IN,org.apache.flink.connector.kafka.sink.KafkaCommittable>
context
- the runtime context.IOException
- for any failure during creation.@Internal public org.apache.flink.connector.kafka.sink.KafkaWriter<IN> restoreWriter(Sink.InitContext context, Collection<org.apache.flink.connector.kafka.sink.KafkaWriterState> recoveredState) throws IOException
StatefulSink
StatefulSink.StatefulSinkWriter
from a recovered state.restoreWriter
in interface StatefulSink<IN,org.apache.flink.connector.kafka.sink.KafkaWriterState>
context
- the runtime context.IOException
- for any failure during creation.@Internal public SimpleVersionedSerializer<org.apache.flink.connector.kafka.sink.KafkaWriterState> getWriterStateSerializer()
StatefulSink
StatefulSink.StatefulSinkWriter.snapshotState(long)
properly. The respective state is used in #restoreWriter(InitContext, Collection)
on recovery.getWriterStateSerializer
in interface StatefulSink<IN,org.apache.flink.connector.kafka.sink.KafkaWriterState>
@VisibleForTesting protected Properties getKafkaProducerConfig()
Copyright © 2014–2024 The Apache Software Foundation. All rights reserved.