Interface TwoPhaseCommittingSink<InputT,CommT>
-
- Type Parameters:
InputT- The type of the sink's inputCommT- The type of the committables.
- All Superinterfaces:
Serializable,Sink<InputT>
@PublicEvolving public interface TwoPhaseCommittingSink<InputT,CommT> extends Sink<InputT>
ASinkfor exactly-once semantics using a two-phase commit protocol. TheSinkconsists of aSinkWriterthat performs the precommits and aCommitterthat actually commits the data. To facilitate the separation theSinkWritercreates committables on checkpoint or end of input and the sends it to theCommitter.The
TwoPhaseCommittingSinkneeds to be serializable. All configuration should be validated eagerly. The respective sink writers and committers are transient and will only be created in the subtasks on the taskmanagers.
-
-
Nested Class Summary
Nested Classes Modifier and Type Interface Description static interfaceTwoPhaseCommittingSink.PrecommittingSinkWriter<InputT,CommT>ASinkWriterthat performs the first part of a two-phase commit protocol.-
Nested classes/interfaces inherited from interface org.apache.flink.api.connector.sink2.Sink
Sink.InitContext
-
-
Method Summary
All Methods Instance Methods Abstract Methods Modifier and Type Method Description Committer<CommT>createCommitter()Creates aCommitterthat permanently makes the previously written data visible throughCommitter.commit(Collection).TwoPhaseCommittingSink.PrecommittingSinkWriter<InputT,CommT>createWriter(Sink.InitContext context)Creates aTwoPhaseCommittingSink.PrecommittingSinkWriterthat creates committables on checkpoint or end of input.SimpleVersionedSerializer<CommT>getCommittableSerializer()Returns the serializer of the committable type.
-
-
-
Method Detail
-
createWriter
TwoPhaseCommittingSink.PrecommittingSinkWriter<InputT,CommT> createWriter(Sink.InitContext context) throws IOException
Creates aTwoPhaseCommittingSink.PrecommittingSinkWriterthat creates committables on checkpoint or end of input.- Specified by:
createWriterin interfaceSink<InputT>- Parameters:
context- the runtime context.- Returns:
- A sink writer for the two-phase commit protocol.
- Throws:
IOException- for any failure during creation.
-
createCommitter
Committer<CommT> createCommitter() throws IOException
Creates aCommitterthat permanently makes the previously written data visible throughCommitter.commit(Collection).- Returns:
- A committer for the two-phase commit protocol.
- Throws:
IOException- for any failure during creation.
-
getCommittableSerializer
SimpleVersionedSerializer<CommT> getCommittableSerializer()
Returns the serializer of the committable type.
-
-