Interface Sink<InputT,CommT,WriterStateT,GlobalCommT>
-
- Type Parameters:
InputT- The type of the sink's inputCommT- The type of information needed to commit data staged by the sinkWriterStateT- The type of the sink writer's stateGlobalCommT- The type of the aggregated committable
- All Superinterfaces:
Serializable
@Deprecated @PublicEvolving public interface Sink<InputT,CommT,WriterStateT,GlobalCommT> extends Serializable
Deprecated.Please useSinkor a derivative.This interface lets the sink developer build a simple sink topology, which could guarantee the exactly once semantics in both batch and stream execution mode if there is aCommitterorGlobalCommitter. 1. TheSinkWriteris responsible for producing the committable. 2. TheCommitteris responsible for committing a single committable. 3. TheGlobalCommitteris responsible for committing an aggregated committable, which we call the global committable. TheGlobalCommitteris always executed with a parallelism of 1. Note: Developers need to ensure the idempotence ofCommitterandGlobalCommitter.A sink must always have a writer, but committer and global committer are each optional and all combinations are valid.
The
Sinkneeds to be serializable. All configuration should be validated eagerly. The respective sink parts are transient and will only be created in the subtasks on the taskmanagers.
-
-
Nested Class Summary
Nested Classes Modifier and Type Interface Description static interfaceSink.InitContextDeprecated.Please migrate toSinkand useSink.InitContext.static interfaceSink.ProcessingTimeServiceDeprecated.Please migrate toSinkand useProcessingTimeService.
-
Method Summary
All Methods Instance Methods Abstract Methods Default Methods Deprecated Methods Modifier and Type Method Description Optional<Committer<CommT>>createCommitter()Deprecated.Creates aCommitterwhich is part of a 2-phase-commit protocol.Optional<GlobalCommitter<CommT,GlobalCommT>>createGlobalCommitter()Deprecated.Creates aGlobalCommitterwhich is part of a 2-phase-commit protocol.SinkWriter<InputT,CommT,WriterStateT>createWriter(Sink.InitContext context, List<WriterStateT> states)Deprecated.Create aSinkWriter.Optional<SimpleVersionedSerializer<CommT>>getCommittableSerializer()Deprecated.Returns the serializer of the committable type.default Collection<String>getCompatibleStateNames()Deprecated.A list of state names of sinks from which the state can be restored.Optional<SimpleVersionedSerializer<GlobalCommT>>getGlobalCommittableSerializer()Deprecated.Returns the serializer of the aggregated committable type.Optional<SimpleVersionedSerializer<WriterStateT>>getWriterStateSerializer()Deprecated.Any stateful sink needs to provide this state serializer and implementSinkWriter.snapshotState(long)properly.
-
-
-
Method Detail
-
createWriter
SinkWriter<InputT,CommT,WriterStateT> createWriter(Sink.InitContext context, List<WriterStateT> states) throws IOException
Deprecated.Create aSinkWriter. If the application is resumed from a checkpoint or savepoint and the sink is stateful, it will receive the corresponding state obtained withSinkWriter.snapshotState(long)and serialized withgetWriterStateSerializer(). If no state exists, the first existing, compatible state specified ingetCompatibleStateNames()will be loaded and passed.- Parameters:
context- the runtime context.states- the writer's previous state.- Returns:
- A sink writer.
- Throws:
IOException- for any failure during creation.- See Also:
SinkWriter.snapshotState(long),getWriterStateSerializer(),getCompatibleStateNames()
-
getWriterStateSerializer
Optional<SimpleVersionedSerializer<WriterStateT>> getWriterStateSerializer()
Deprecated.Any stateful sink needs to provide this state serializer and implementSinkWriter.snapshotState(long)properly. The respective state is used increateWriter(InitContext, List)on recovery.- Returns:
- the serializer of the writer's state type.
-
createCommitter
Optional<Committer<CommT>> createCommitter() throws IOException
Deprecated.Creates aCommitterwhich is part of a 2-phase-commit protocol. TheSinkWritercreates committables throughSinkWriter.prepareCommit(boolean)in the first phase. The committables are then passed to this committer and persisted withCommitter.commit(List). If a committer is returned, the sink must also return agetCommittableSerializer().- Returns:
- A committer for the 2-phase-commit protocol.
- Throws:
IOException- for any failure during creation.
-
createGlobalCommitter
Optional<GlobalCommitter<CommT,GlobalCommT>> createGlobalCommitter() throws IOException
Deprecated.Creates aGlobalCommitterwhich is part of a 2-phase-commit protocol. TheSinkWritercreates committables throughSinkWriter.prepareCommit(boolean)in the first phase. The committables are then passed to the Committer and persisted withCommitter.commit(List). The committables are also passed to thisGlobalCommitterof which only a single instance exists. If a global committer is returned, the sink must also return agetCommittableSerializer()andgetGlobalCommittableSerializer().- Returns:
- A global committer for the 2-phase-commit protocol.
- Throws:
IOException- for any failure during creation.
-
getCommittableSerializer
Optional<SimpleVersionedSerializer<CommT>> getCommittableSerializer()
Deprecated.Returns the serializer of the committable type. The serializer is required iff the sink has aCommitterorGlobalCommitter.
-
getGlobalCommittableSerializer
Optional<SimpleVersionedSerializer<GlobalCommT>> getGlobalCommittableSerializer()
Deprecated.Returns the serializer of the aggregated committable type. The serializer is required iff the sink has aGlobalCommitter.
-
getCompatibleStateNames
default Collection<String> getCompatibleStateNames()
Deprecated.A list of state names of sinks from which the state can be restored. For example, the newFileSinkcan resume from the state of an oldStreamingFileSinkas a drop-in replacement when resuming from a checkpoint/savepoint.
-
-