Interface StatefulSink<InputT,WriterStateT>
-
- Type Parameters:
InputT- The type of the sink's inputWriterStateT- The type of the sink writer's state
- All Superinterfaces:
Serializable,Sink<InputT>
@PublicEvolving public interface StatefulSink<InputT,WriterStateT> extends Sink<InputT>
ASinkwith a statefulSinkWriter.The
StatefulSinkneeds to be serializable. All configuration should be validated eagerly. The respective sink writers are transient and will only be created in the subtasks on the taskmanagers.
-
-
Nested Class Summary
Nested Classes Modifier and Type Interface Description static interfaceStatefulSink.StatefulSinkWriter<InputT,WriterStateT>ASinkWriterwhose state needs to be checkpointed.static interfaceStatefulSink.WithCompatibleStateA mix-in forStatefulSinkthat allows users to migrate from a sink with a compatible state to this sink.-
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 StatefulSink.StatefulSinkWriter<InputT,WriterStateT>createWriter(Sink.InitContext context)Create aStatefulSink.StatefulSinkWriter.SimpleVersionedSerializer<WriterStateT>getWriterStateSerializer()Any stateful sink needs to provide this state serializer and implementStatefulSink.StatefulSinkWriter.snapshotState(long)properly.StatefulSink.StatefulSinkWriter<InputT,WriterStateT>restoreWriter(Sink.InitContext context, Collection<WriterStateT> recoveredState)Create aStatefulSink.StatefulSinkWriterfrom a recovered state.
-
-
-
Method Detail
-
createWriter
StatefulSink.StatefulSinkWriter<InputT,WriterStateT> createWriter(Sink.InitContext context) throws IOException
Create aStatefulSink.StatefulSinkWriter.- Specified by:
createWriterin interfaceSink<InputT>- Parameters:
context- the runtime context.- Returns:
- A sink writer.
- Throws:
IOException- for any failure during creation.
-
restoreWriter
StatefulSink.StatefulSinkWriter<InputT,WriterStateT> restoreWriter(Sink.InitContext context, Collection<WriterStateT> recoveredState) throws IOException
Create aStatefulSink.StatefulSinkWriterfrom a recovered state.- Parameters:
context- the runtime context.- Returns:
- A sink writer.
- Throws:
IOException- for any failure during creation.
-
getWriterStateSerializer
SimpleVersionedSerializer<WriterStateT> getWriterStateSerializer()
Any stateful sink needs to provide this state serializer and implementStatefulSink.StatefulSinkWriter.snapshotState(long)properly. The respective state is used inrestoreWriter(InitContext, Collection)on recovery.- Returns:
- the serializer of the writer's state type.
-
-