Type
- The type of the messages created by the source.UId
- The type of the unique IDs which are consistent across sessions.SessionId
- The type of the IDs that are used for acknowledging elements (ids valid during
session).@PublicEvolving public abstract class MultipleIdsMessageAcknowledgingSourceBase<Type,UId,SessionId> extends MessageAcknowledgingSourceBase<Type,UId>
MessageAcknowledgingSourceBase
, this source handles two
types of ids:
Session ids are used to acknowledge messages in a session. When a checkpoint is restored, unacknowledged messages are redelivered. Duplicates are detected using the unique message ids which are checkpointed.
SourceFunction.SourceContext<T>
Modifier and Type | Field and Description |
---|---|
protected List<SessionId> |
sessionIds |
protected Deque<Tuple2<Long,List<SessionId>>> |
sessionIdsPerSnapshot |
pendingCheckpoints
Modifier | Constructor and Description |
---|---|
protected |
MultipleIdsMessageAcknowledgingSourceBase(Class<UId> idClass)
Creates a new MessageAcknowledgingSourceBase for IDs of the given type.
|
protected |
MultipleIdsMessageAcknowledgingSourceBase(TypeInformation<UId> idTypeInfo)
Creates a new MessageAcknowledgingSourceBase for IDs of the given type.
|
Modifier and Type | Method and Description |
---|---|
protected void |
acknowledgeIDs(long checkpointId,
Set<UId> uniqueIds)
Acknowledges the session ids.
|
protected abstract void |
acknowledgeSessionIDs(List<SessionId> sessionIds)
Acknowledges the session ids.
|
void |
close()
Tear-down method for the user code.
|
void |
open(Configuration parameters)
Initialization method for the function.
|
void |
snapshotState(FunctionSnapshotContext context)
This method is called when a snapshot for a checkpoint is requested.
|
addId, initializeState, notifyCheckpointAborted, notifyCheckpointComplete
getIterationRuntimeContext, getRuntimeContext, setRuntimeContext
clone, equals, finalize, getClass, hashCode, notify, notifyAll, toString, wait, wait, wait
cancel, run
protected MultipleIdsMessageAcknowledgingSourceBase(Class<UId> idClass)
idClass
- The class of the message ID type, used to create a serializer for the message
IDs.protected MultipleIdsMessageAcknowledgingSourceBase(TypeInformation<UId> idTypeInfo)
idTypeInfo
- The type information of the message ID type, used to create a serializer
for the message IDs.public void open(Configuration parameters) throws Exception
RichFunction
The configuration object passed to the function can be used for configuration and initialization. The configuration contains all parameters that were configured on the function in the program composition.
public class MyFilter extends RichFilterFunction<String> {
private String searchString;
public void open(Configuration parameters) {
this.searchString = parameters.getString("foo");
}
public boolean filter(String value) {
return value.equals(searchString);
}
}
By default, this method does nothing.
open
in interface RichFunction
open
in class AbstractRichFunction
parameters
- The configuration containing the parameters attached to the contract.Exception
- Implementations may forward exceptions, which are caught by the runtime.
When the runtime catches an exception, it aborts the task and lets the fail-over logic
decide whether to retry the task execution.Configuration
public void close() throws Exception
RichFunction
This method can be used for clean up work.
close
in interface RichFunction
close
in class MessageAcknowledgingSourceBase<Type,UId>
Exception
- Implementations may forward exceptions, which are caught by the runtime.
When the runtime catches an exception, it aborts the task and lets the fail-over logic
decide whether to retry the task execution.protected final void acknowledgeIDs(long checkpointId, Set<UId> uniqueIds)
acknowledgeIDs
in class MessageAcknowledgingSourceBase<Type,UId>
checkpointId
- The id of the current checkout to acknowledge ids for.uniqueIds
- The checkpointed unique ids which are ignored here. They only serve as a
means of de-duplicating messages when the acknowledgment after a checkpoint fails.protected abstract void acknowledgeSessionIDs(List<SessionId> sessionIds)
sessionIds
- The message ids for this session.public void snapshotState(FunctionSnapshotContext context) throws Exception
CheckpointedFunction
FunctionInitializationContext
when the Function was initialized, or offered now by FunctionSnapshotContext
itself.snapshotState
in interface CheckpointedFunction
snapshotState
in class MessageAcknowledgingSourceBase<Type,UId>
context
- the context for drawing a snapshot of the operatorException
- Thrown, if state could not be created ot restored.Copyright © 2014–2024 The Apache Software Foundation. All rights reserved.