@Internal public class StreamingRuntimeContext extends AbstractRuntimeUDFContext
RuntimeContext
, for streaming
operators.Constructor and Description |
---|
StreamingRuntimeContext(AbstractStreamOperator<?> operator,
Environment env,
Map<String,Accumulator<?,?>> accumulators) |
StreamingRuntimeContext(Environment env,
Map<String,Accumulator<?,?>> accumulators,
OperatorMetricGroup operatorMetricGroup,
OperatorID operatorID,
ProcessingTimeService processingTimeService,
KeyedStateStore keyedStateStore,
ExternalResourceInfoProvider externalResourceInfoProvider) |
Modifier and Type | Method and Description |
---|---|
<IN,ACC,OUT> |
getAggregatingState(AggregatingStateDescriptor<IN,ACC,OUT> stateProperties)
Gets a handle to the system's key/value aggregating state.
|
<IN,ACC,OUT> |
getAggregatingState(AggregatingStateDescriptor<IN,ACC,OUT> stateProperties) |
<RT> List<RT> |
getBroadcastVariable(String name)
Returns the result bound to the broadcast variable identified by the given
name . |
<T,C> C |
getBroadcastVariableWithInitializer(String name,
BroadcastVariableInitializer<T,C> initializer)
Returns the result bound to the broadcast variable identified by the given
name . |
Set<ExternalResourceInfo> |
getExternalResourceInfos(String resourceName)
Get the specific external resource information by the resourceName.
|
GlobalAggregateManager |
getGlobalAggregateManager()
Returns the global aggregate manager for the current job.
|
InputSplitProvider |
getInputSplitProvider()
Returns the input split provider associated with the operator.
|
Configuration |
getJobConfiguration() |
JobType |
getJobType() |
<T> ListState<T> |
getListState(ListStateDescriptor<T> stateProperties)
Gets a handle to the system's key/value list state.
|
<T> ListState<T> |
getListState(ListStateDescriptor<T> stateProperties) |
<UK,UV> MapState<UK,UV> |
getMapState(MapStateDescriptor<UK,UV> stateProperties)
Gets a handle to the system's key/value map state.
|
<UK,UV> MapState<UK,UV> |
getMapState(MapStateDescriptor<UK,UV> stateProperties) |
String |
getOperatorUniqueID()
Returned value is guaranteed to be unique between operators within the same job and to be
stable and the same across job submissions.
|
ProcessingTimeService |
getProcessingTimeService() |
<T> ReducingState<T> |
getReducingState(ReducingStateDescriptor<T> stateProperties)
Gets a handle to the system's key/value reducing state.
|
<T> ReducingState<T> |
getReducingState(ReducingStateDescriptor<T> stateProperties) |
<T> ValueState<T> |
getState(ValueStateDescriptor<T> stateProperties)
Gets a handle to the system's key/value state.
|
TaskManagerRuntimeInfo |
getTaskManagerRuntimeInfo()
Returns the task manager runtime info of the task manager running this stream task.
|
<T> ValueState<T> |
getValueState(ValueStateDescriptor<T> stateProperties) |
boolean |
hasBroadcastVariable(String name)
Tests for the existence of the broadcast variable identified by the given
name . |
boolean |
isCheckpointingEnabled()
Returns true if checkpointing is enabled for the running job.
|
void |
setKeyedStateStore(KeyedStateStore keyedStateStore) |
void |
setKeyedStateStoreV2(KeyedStateStoreV2 keyedStateStoreV2) |
addAccumulator, createSerializer, getAccumulator, getAllocationIDAsString, getDistributedCache, getDoubleCounter, getGlobalJobParameters, getHistogram, getIntCounter, getJobInfo, getLongCounter, getMetricGroup, getTaskInfo, getUserCodeClassLoader, isObjectReuseEnabled, registerUserCodeClassLoaderReleaseHookIfAbsent
@VisibleForTesting public StreamingRuntimeContext(AbstractStreamOperator<?> operator, Environment env, Map<String,Accumulator<?,?>> accumulators)
public StreamingRuntimeContext(Environment env, Map<String,Accumulator<?,?>> accumulators, OperatorMetricGroup operatorMetricGroup, OperatorID operatorID, ProcessingTimeService processingTimeService, @Nullable KeyedStateStore keyedStateStore, ExternalResourceInfoProvider externalResourceInfoProvider)
public void setKeyedStateStore(@Nullable KeyedStateStore keyedStateStore)
public void setKeyedStateStoreV2(@Nullable KeyedStateStoreV2 keyedStateStoreV2)
public InputSplitProvider getInputSplitProvider()
public ProcessingTimeService getProcessingTimeService()
public GlobalAggregateManager getGlobalAggregateManager()
public String getOperatorUniqueID()
This operation is currently only supported in Streaming (DataStream) contexts.
public TaskManagerRuntimeInfo getTaskManagerRuntimeInfo()
public Configuration getJobConfiguration()
public JobType getJobType()
public Set<ExternalResourceInfo> getExternalResourceInfos(String resourceName)
RuntimeContext
resourceName
- of the required external resourcepublic boolean hasBroadcastVariable(String name)
RuntimeContext
name
.name
- The name under which the broadcast variable is registered;public <RT> List<RT> getBroadcastVariable(String name)
RuntimeContext
name
.
IMPORTANT: The broadcast variable data structure is shared between the parallel tasks on one machine. Any access that modifies its internal state needs to be manually synchronized by the caller.
name
- The name under which the broadcast variable is registered;public <T,C> C getBroadcastVariableWithInitializer(String name, BroadcastVariableInitializer<T,C> initializer)
RuntimeContext
name
. The
broadcast variable is returned as a shared data structure that is initialized with the given
BroadcastVariableInitializer
.
IMPORTANT: The broadcast variable data structure is shared between the parallel tasks on one machine. Any access that modifies its internal state needs to be manually synchronized by the caller.
name
- The name under which the broadcast variable is registered;initializer
- The initializer that creates the shared data structure of the broadcast
variable from the sequence of elements.public <T> ValueState<T> getState(ValueStateDescriptor<T> stateProperties)
RuntimeContext
Because the scope of each value is the key of the currently processed element, and the elements are distributed by the Flink runtime, the system can transparently scale out and redistribute the state and KeyedStream.
The following code example shows how to implement a continuous counter that counts how many times elements of a certain key occur, and emits an updated count for that element on each occurrence.
DataStream<MyType> stream = ...;
KeyedStream<MyType> keyedStream = stream.keyBy("id");
keyedStream.map(new RichMapFunction<MyType, Tuple2<MyType, Long>>() {
private ValueState<Long> state;
public void open(OpenContext ctx) {
state = getRuntimeContext().getState(
new ValueStateDescriptor<Long>("count", LongSerializer.INSTANCE, 0L));
}
public Tuple2<MyType, Long> map(MyType value) {
long count = state.value() + 1;
state.update(count);
return new Tuple2<>(value, count);
}
});
getState
in interface RuntimeContext
getState
in class AbstractRuntimeUDFContext
T
- The type of value stored in the state.stateProperties
- The descriptor defining the properties of the stats.public <T> ListState<T> getListState(ListStateDescriptor<T> stateProperties)
RuntimeContext
RuntimeContext.getState(ValueStateDescriptor)
, but is optimized for state that holds
lists. One can add elements to the list, or retrieve the list as a whole.
This state is only accessible if the function is executed on a KeyedStream.
DataStream<MyType> stream = ...;
KeyedStream<MyType> keyedStream = stream.keyBy("id");
keyedStream.map(new RichFlatMapFunction<MyType, List<MyType>>() {
private ListState<MyType> state;
public void open(OpenContext ctx) {
state = getRuntimeContext().getListState(
new ListStateDescriptor<>("myState", MyType.class));
}
public void flatMap(MyType value, Collector<MyType> out) {
if (value.isDivider()) {
for (MyType t : state.get()) {
out.collect(t);
}
} else {
state.add(value);
}
}
});
getListState
in interface RuntimeContext
getListState
in class AbstractRuntimeUDFContext
T
- The type of value stored in the state.stateProperties
- The descriptor defining the properties of the stats.public <T> ReducingState<T> getReducingState(ReducingStateDescriptor<T> stateProperties)
RuntimeContext
RuntimeContext.getState(ValueStateDescriptor)
, but is optimized for state that
aggregates values.
This state is only accessible if the function is executed on a KeyedStream.
DataStream<MyType> stream = ...;
KeyedStream<MyType> keyedStream = stream.keyBy("id");
keyedStream.map(new RichMapFunction<MyType, List<MyType>>() {
private ReducingState<Long> state;
public void open(OpenContext ctx) {
state = getRuntimeContext().getReducingState(
new ReducingStateDescriptor<>("sum", (a, b) -> a + b, Long.class));
}
public Tuple2<MyType, Long> map(MyType value) {
state.add(value.count());
return new Tuple2<>(value, state.get());
}
});
getReducingState
in interface RuntimeContext
getReducingState
in class AbstractRuntimeUDFContext
T
- The type of value stored in the state.stateProperties
- The descriptor defining the properties of the stats.public <IN,ACC,OUT> AggregatingState<IN,OUT> getAggregatingState(AggregatingStateDescriptor<IN,ACC,OUT> stateProperties)
RuntimeContext
RuntimeContext.getState(ValueStateDescriptor)
, but is optimized for state that
aggregates values with different types.
This state is only accessible if the function is executed on a KeyedStream.
DataStream<MyType> stream = ...;
KeyedStream<MyType> keyedStream = stream.keyBy("id");
AggregateFunction<...> aggregateFunction = ...
keyedStream.map(new RichMapFunction<MyType, List<MyType>>() {
private AggregatingState<MyType, Long> state;
public void open(OpenContext ctx) {
state = getRuntimeContext().getAggregatingState(
new AggregatingStateDescriptor<>("sum", aggregateFunction, Long.class));
}
public Tuple2<MyType, Long> map(MyType value) {
state.add(value);
return new Tuple2<>(value, state.get());
}
});
getAggregatingState
in interface RuntimeContext
getAggregatingState
in class AbstractRuntimeUDFContext
IN
- The type of the values that are added to the state.ACC
- The type of the accumulator (intermediate aggregation state).OUT
- The type of the values that are returned from the state.stateProperties
- The descriptor defining the properties of the stats.public <UK,UV> MapState<UK,UV> getMapState(MapStateDescriptor<UK,UV> stateProperties)
RuntimeContext
RuntimeContext.getState(ValueStateDescriptor)
, but is optimized for state that is
composed of user-defined key-value pairs
This state is only accessible if the function is executed on a KeyedStream.
DataStream<MyType> stream = ...;
KeyedStream<MyType> keyedStream = stream.keyBy("id");
keyedStream.map(new RichMapFunction<MyType, List<MyType>>() {
private MapState<MyType, Long> state;
public void open(OpenContext ctx) {
state = getRuntimeContext().getMapState(
new MapStateDescriptor<>("sum", MyType.class, Long.class));
}
public Tuple2<MyType, Long> map(MyType value) {
return new Tuple2<>(value, state.get(value));
}
});
getMapState
in interface RuntimeContext
getMapState
in class AbstractRuntimeUDFContext
UK
- The type of the user keys stored in the state.UV
- The type of the user values stored in the state.stateProperties
- The descriptor defining the properties of the stats.public <T> ValueState<T> getValueState(ValueStateDescriptor<T> stateProperties)
public <T> ListState<T> getListState(ListStateDescriptor<T> stateProperties)
public <UK,UV> MapState<UK,UV> getMapState(MapStateDescriptor<UK,UV> stateProperties)
public <T> ReducingState<T> getReducingState(ReducingStateDescriptor<T> stateProperties)
public <IN,ACC,OUT> AggregatingState<IN,OUT> getAggregatingState(AggregatingStateDescriptor<IN,ACC,OUT> stateProperties)
public boolean isCheckpointingEnabled()
Copyright © 2014–2024 The Apache Software Foundation. All rights reserved.