ProcessorContextImpl¶
ProcessorContextImpl is an AbstractProcessorContext.
Creating Instance¶
ProcessorContextImpl takes the following to be created:
ProcessorContextImpl is created when:
ActiveTaskCreatoris requested to createTasksStandbyTaskCreatoris requested to createTasks
ProcessorStateManager¶
ProcessorContextImpl is given a ProcessorStateManager when created.
Scheduling Recurring Action¶
Cancellable schedule(
Duration interval,
PunctuationType type,
Punctuator callback)
schedule converts the interval to milliseconds and requests the StreamTask to schedule the given Punctuator.
schedule makes sure that the interval is at least 1 ms or throws an IllegalArgumentException:
The minimum supported scheduling interval is 1 millisecond.
schedule is part of the ProcessorContext abstraction.
Forwarding Record Downstream¶
ProcessorContextImpl is associated with a ProcessorNode known as the current node.
forward (and forwardInternal in particular) uses the current ProcessorNode to process a given record.
ProcessorNodes are associated with child ProcessorNodes known as children.
forward¶
void forward(
K key,
V value)
void forward(
K key,
V value,
To to)
void forward(
Record<K, V> record)
void forward(
Record<K, V> record,
String childName)
forward forwardInternal to the child or all the children nodes of the current ProcessorNode.
forward throws an UnsupportedOperationException for a TaskType.STANDBY task:
this should not happen: forward() is not supported in standby tasks.
forward is part of the ProcessorContext abstraction.
forwardInternal¶
void forwardInternal(
ProcessorNode<K, V, ?, ?> child,
Record<K, V> record)
forwardInternal sets the current node to be the given child ProcessorNode that is in turn requested to process the record.
If the child node is terminal (no children), forwardInternal requests the StreamTask to maybeRecordE2ELatency.