Interface RuntimeContext
-
- All Known Subinterfaces:
IterationRuntimeContext
- All Known Implementing Classes:
AbstractRuntimeUDFContext,RuntimeUDFContext
@Public public interface RuntimeContextA RuntimeContext contains information about the context in which functions are executed. Each parallel instance of the function will have a context through which it can access static contextual information (such as the current parallelism) and other constructs like accumulators and broadcast variables.A function can, during runtime, obtain the RuntimeContext via a call to
AbstractRichFunction.getRuntimeContext().
-
-
Method Summary
All Methods Instance Methods Abstract Methods Default Methods Deprecated Methods Modifier and Type Method Description <V,A extends Serializable>
voidaddAccumulator(String name, Accumulator<V,A> accumulator)Add this accumulator.<T> TypeSerializer<T>createSerializer(TypeInformation<T> typeInformation)Create a serializer for a given type.<V,A extends Serializable>
Accumulator<V,A>getAccumulator(String name)Get an existing accumulator object.<IN,ACC,OUT>
org.apache.flink.api.common.state.AggregatingState<IN,OUT>getAggregatingState(AggregatingStateDescriptor<IN,ACC,OUT> stateProperties)Gets a handle to the system's key/value aggregating state.default intgetAttemptNumber()Deprecated.This method is deprecated since Flink 1.19.<RT> List<RT>getBroadcastVariable(String name)Returns the result bound to the broadcast variable identified by the givenname.<T,C>
CgetBroadcastVariableWithInitializer(String name, BroadcastVariableInitializer<T,C> initializer)Returns the result bound to the broadcast variable identified by the givenname.DistributedCachegetDistributedCache()Returns theDistributedCacheto get the local temporary file copies of files otherwise not locally accessible.DoubleCountergetDoubleCounter(String name)Convenience function to create a counter object for doubles.ExecutionConfiggetExecutionConfig()Deprecated.This method has been deprecated and will be removed in the upcoming FLINK major version FLINK-2.0.Set<ExternalResourceInfo>getExternalResourceInfos(String resourceName)Get the specific external resource information by the resourceName.Map<String,String>getGlobalJobParameters()Get global job parameters.HistogramgetHistogram(String name)Convenience function to create a counter object for histograms.default intgetIndexOfThisSubtask()Deprecated.This method is deprecated since Flink 1.19.IntCountergetIntCounter(String name)Convenience function to create a counter object for integers.default JobIDgetJobId()Deprecated.This method is deprecated since Flink 1.19.JobInfogetJobInfo()Get the meta information of current job.<T> org.apache.flink.api.common.state.ListState<T>getListState(ListStateDescriptor<T> stateProperties)Gets a handle to the system's key/value list state.LongCountergetLongCounter(String name)Convenience function to create a counter object for longs.<UK,UV>
org.apache.flink.api.common.state.MapState<UK,UV>getMapState(MapStateDescriptor<UK,UV> stateProperties)Gets a handle to the system's key/value map state.default intgetMaxNumberOfParallelSubtasks()Deprecated.This method is deprecated since Flink 1.19.org.apache.flink.metrics.groups.OperatorMetricGroupgetMetricGroup()Returns the metric group for this parallel subtask.default intgetNumberOfParallelSubtasks()Deprecated.This method is deprecated since Flink 1.19.<T> org.apache.flink.api.common.state.ReducingState<T>getReducingState(ReducingStateDescriptor<T> stateProperties)Gets a handle to the system's key/value reducing state.<T> org.apache.flink.api.common.state.ValueState<T>getState(ValueStateDescriptor<T> stateProperties)Gets a handle to the system's key/value state.TaskInfogetTaskInfo()Get the meta information of current task.default StringgetTaskName()Deprecated.This method is deprecated since Flink 1.19.default StringgetTaskNameWithSubtasks()Deprecated.This method is deprecated since Flink 1.19.ClassLoadergetUserCodeClassLoader()Gets the ClassLoader to load classes that are not in system's classpath, but are part of the jar file of a user job.booleanhasBroadcastVariable(String name)Tests for the existence of the broadcast variable identified by the givenname.booleanisObjectReuseEnabled()Check if object reuse is enabled.voidregisterUserCodeClassLoaderReleaseHookIfAbsent(String releaseHookName, Runnable releaseHook)Registers a custom hook for the user code class loader release.
-
-
-
Method Detail
-
getJobId
@Deprecated default JobID getJobId()
Deprecated.This method is deprecated since Flink 1.19. All metadata about the job should be provided uniformly bygetJobInfo().The ID of the current job. Note that Job ID can change in particular upon manual restart. The returned ID should NOT be used for any job management tasks.
-
getTaskName
@Deprecated default String getTaskName()
Deprecated.This method is deprecated since Flink 1.19. All metadata about the task should be provided uniformly bygetTaskInfo().Returns the name of the task in which the UDF runs, as assigned during plan construction.- Returns:
- The name of the task in which the UDF runs.
- See Also:
- FLIP-382: Unify the Provision of Diverse Metadata for Context-like APIs
-
getMetricGroup
@PublicEvolving org.apache.flink.metrics.groups.OperatorMetricGroup getMetricGroup()
Returns the metric group for this parallel subtask.- Returns:
- The metric group for this parallel subtask.
-
getNumberOfParallelSubtasks
@Deprecated default int getNumberOfParallelSubtasks()
Deprecated.This method is deprecated since Flink 1.19. All metadata about the task should be provided uniformly bygetTaskInfo().Gets the parallelism with which the parallel task runs.- Returns:
- The parallelism with which the parallel task runs.
- See Also:
- FLIP-382: Unify the Provision of Diverse Metadata for Context-like APIs
-
getMaxNumberOfParallelSubtasks
@Deprecated @PublicEvolving default int getMaxNumberOfParallelSubtasks()
Deprecated.This method is deprecated since Flink 1.19. All metadata about the task should be provided uniformly bygetTaskInfo().Gets the number of max-parallelism with which the parallel task runs.- Returns:
- The max-parallelism with which the parallel task runs.
- See Also:
- FLIP-382: Unify the Provision of Diverse Metadata for Context-like APIs
-
getIndexOfThisSubtask
@Deprecated default int getIndexOfThisSubtask()
Deprecated.This method is deprecated since Flink 1.19. All metadata about the task should be provided uniformly bygetTaskInfo().Gets the number of this parallel subtask. The numbering starts from 0 and goes up to parallelism-1.- Returns:
- The index of the parallel subtask.
- See Also:
- FLIP-382: Unify the Provision of Diverse Metadata for Context-like APIs
-
getAttemptNumber
@Deprecated default int getAttemptNumber()
Deprecated.This method is deprecated since Flink 1.19. All metadata about the task should be provided uniformly bygetTaskInfo().Gets the attempt number of this parallel subtask. First attempt is numbered 0.- Returns:
- Attempt number of the subtask.
- See Also:
- FLIP-382: Unify the Provision of Diverse Metadata for Context-like APIs
-
getTaskNameWithSubtasks
@Deprecated default String getTaskNameWithSubtasks()
Deprecated.This method is deprecated since Flink 1.19. All metadata about the task should be provided uniformly bygetTaskInfo().Returns the name of the task, appended with the subtask indicator, such as "MyTask (3/6)#1", where 3 would be (task index + 1), and 6 would be task parallelism, and 1 would be attempt number.- Returns:
- The name of the task, with subtask indicator.
- See Also:
- FLIP-382: Unify the Provision of Diverse Metadata for Context-like APIs
-
getExecutionConfig
@Deprecated ExecutionConfig getExecutionConfig()
Deprecated.This method has been deprecated and will be removed in the upcoming FLINK major version FLINK-2.0. Users relying on this method should migrate to alternative getter methods, such asgetGlobalJobParameters()orisObjectReuseEnabled().Returns theExecutionConfigfor the currently executing job.
-
createSerializer
@PublicEvolving <T> TypeSerializer<T> createSerializer(TypeInformation<T> typeInformation)
Create a serializer for a given type.- Parameters:
typeInformation- the type information of the object to be serialized- Returns:
- the serializer for the given type
-
getGlobalJobParameters
@PublicEvolving Map<String,String> getGlobalJobParameters()
Get global job parameters.- Returns:
- the global job parameters
-
isObjectReuseEnabled
@PublicEvolving boolean isObjectReuseEnabled()
Check if object reuse is enabled.- Returns:
- true if object reuse is enabled, false otherwise
-
getUserCodeClassLoader
ClassLoader getUserCodeClassLoader()
Gets the ClassLoader to load classes that are not in system's classpath, but are part of the jar file of a user job.- Returns:
- The ClassLoader for user code classes.
-
registerUserCodeClassLoaderReleaseHookIfAbsent
@PublicEvolving void registerUserCodeClassLoaderReleaseHookIfAbsent(String releaseHookName, Runnable releaseHook)
Registers a custom hook for the user code class loader release.The release hook is executed just before the user code class loader is being released. Registration only happens if no hook has been registered under this name already.
- Parameters:
releaseHookName- name of the release hook.releaseHook- release hook which is executed just before the user code class loader is being released
-
addAccumulator
<V,A extends Serializable> void addAccumulator(String name, Accumulator<V,A> accumulator)
Add this accumulator. Throws an exception if the accumulator already exists in the same Task. Note that the Accumulator name must have an unique name across the Flink job. Otherwise you will get an error when incompatible accumulators from different Tasks are combined at the JobManager upon job completion.
-
getAccumulator
<V,A extends Serializable> Accumulator<V,A> getAccumulator(String name)
Get an existing accumulator object. The accumulator must have been added previously in this local runtime context.Throws an exception if the accumulator does not exist or if the accumulator exists, but with different type.
-
getIntCounter
@PublicEvolving IntCounter getIntCounter(String name)
Convenience function to create a counter object for integers.
-
getLongCounter
@PublicEvolving LongCounter getLongCounter(String name)
Convenience function to create a counter object for longs.
-
getDoubleCounter
@PublicEvolving DoubleCounter getDoubleCounter(String name)
Convenience function to create a counter object for doubles.
-
getHistogram
@PublicEvolving Histogram getHistogram(String name)
Convenience function to create a counter object for histograms.
-
getExternalResourceInfos
@PublicEvolving Set<ExternalResourceInfo> getExternalResourceInfos(String resourceName)
Get the specific external resource information by the resourceName.- Parameters:
resourceName- of the required external resource- Returns:
- information set of the external resource identified by the resourceName
-
hasBroadcastVariable
@PublicEvolving boolean hasBroadcastVariable(String name)
Tests for the existence of the broadcast variable identified by the givenname.- Parameters:
name- The name under which the broadcast variable is registered;- Returns:
- Whether a broadcast variable exists for the given name.
-
getBroadcastVariable
<RT> List<RT> getBroadcastVariable(String name)
Returns the result bound to the broadcast variable identified by the givenname.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.
- Parameters:
name- The name under which the broadcast variable is registered;- Returns:
- The broadcast variable, materialized as a list of elements.
-
getBroadcastVariableWithInitializer
<T,C> C getBroadcastVariableWithInitializer(String name, BroadcastVariableInitializer<T,C> initializer)
Returns the result bound to the broadcast variable identified by the givenname. The broadcast variable is returned as a shared data structure that is initialized with the givenBroadcastVariableInitializer.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.
- Parameters:
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.- Returns:
- The broadcast variable, materialized as a list of elements.
-
getDistributedCache
DistributedCache getDistributedCache()
Returns theDistributedCacheto get the local temporary file copies of files otherwise not locally accessible.- Returns:
- The distributed cache of the worker executing this instance.
-
getState
@PublicEvolving <T> org.apache.flink.api.common.state.ValueState<T> getState(ValueStateDescriptor<T> stateProperties)
Gets a handle to the system's key/value state. The key/value state is only accessible if the function is executed on a KeyedStream. On each access, the state exposes the value for the key of the element currently processed by the function. Each function may have multiple partitioned states, addressed with different names.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(Configuration cfg) { 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); } });- Type Parameters:
T- The type of value stored in the state.- Parameters:
stateProperties- The descriptor defining the properties of the stats.- Returns:
- The partitioned state object.
- Throws:
UnsupportedOperationException- Thrown, if no partitioned state is available for the function (function is not part of a KeyedStream).
-
getListState
@PublicEvolving <T> org.apache.flink.api.common.state.ListState<T> getListState(ListStateDescriptor<T> stateProperties)
Gets a handle to the system's key/value list state. This state is similar to the state accessed viagetState(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(Configuration cfg) { 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); } } });- Type Parameters:
T- The type of value stored in the state.- Parameters:
stateProperties- The descriptor defining the properties of the stats.- Returns:
- The partitioned state object.
- Throws:
UnsupportedOperationException- Thrown, if no partitioned state is available for the function (function is not part os a KeyedStream).
-
getReducingState
@PublicEvolving <T> org.apache.flink.api.common.state.ReducingState<T> getReducingState(ReducingStateDescriptor<T> stateProperties)
Gets a handle to the system's key/value reducing state. This state is similar to the state accessed viagetState(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(Configuration cfg) { 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()); } });- Type Parameters:
T- The type of value stored in the state.- Parameters:
stateProperties- The descriptor defining the properties of the stats.- Returns:
- The partitioned state object.
- Throws:
UnsupportedOperationException- Thrown, if no partitioned state is available for the function (function is not part of a KeyedStream).
-
getAggregatingState
@PublicEvolving <IN,ACC,OUT> org.apache.flink.api.common.state.AggregatingState<IN,OUT> getAggregatingState(AggregatingStateDescriptor<IN,ACC,OUT> stateProperties)
Gets a handle to the system's key/value aggregating state. This state is similar to the state accessed viagetState(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(Configuration cfg) { 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()); } });- Type Parameters:
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.- Parameters:
stateProperties- The descriptor defining the properties of the stats.- Returns:
- The partitioned state object.
- Throws:
UnsupportedOperationException- Thrown, if no partitioned state is available for the function (function is not part of a KeyedStream).
-
getMapState
@PublicEvolving <UK,UV> org.apache.flink.api.common.state.MapState<UK,UV> getMapState(MapStateDescriptor<UK,UV> stateProperties)
Gets a handle to the system's key/value map state. This state is similar to the state accessed viagetState(ValueStateDescriptor), but is optimized for state that is composed of user-defined key-value pairsThis 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(Configuration cfg) { state = getRuntimeContext().getMapState( new MapStateDescriptor<>("sum", MyType.class, Long.class)); } public Tuple2<MyType, Long> map(MyType value) { return new Tuple2<>(value, state.get(value)); } });- Type Parameters:
UK- The type of the user keys stored in the state.UV- The type of the user values stored in the state.- Parameters:
stateProperties- The descriptor defining the properties of the stats.- Returns:
- The partitioned state object.
- Throws:
UnsupportedOperationException- Thrown, if no partitioned state is available for the function (function is not part of a KeyedStream).
-
getJobInfo
@PublicEvolving JobInfo getJobInfo()
Get the meta information of current job.- Returns:
- the job meta information.
-
getTaskInfo
@PublicEvolving TaskInfo getTaskInfo()
Get the meta information of current task.- Returns:
- the task meta information.
-
-