Interface OperatorStateStore


  • @PublicEvolving
    public interface OperatorStateStore
    This interface contains methods for registering operator state with a managed store.
    • Method Detail

      • getBroadcastState

        <K,​V> org.apache.flink.api.common.state.BroadcastState<K,​V> getBroadcastState​(MapStateDescriptor<K,​V> stateDescriptor)
                                                                                           throws Exception
        Creates (or restores) a broadcast state. This type of state can only be created to store the state of a BroadcastStream. Each state is registered under a unique name. The provided serializer is used to de/serialize the state in case of checkpointing (snapshot/restore). The returned broadcast state has key-value format.

        CAUTION: the user has to guarantee that all task instances store the same elements in this type of state.

        Each operator instance individually maintains and stores elements in the broadcast state. The fact that the incoming stream is a broadcast one guarantees that all instances see all the elements. Upon recovery or re-scaling, the same state is given to each of the instances. To avoid hotspots, each task reads its previous partition, and if there are more tasks (scale up), then the new instances read from the old instances in a round robin fashion. This is why each instance has to guarantee that it stores the same elements as the rest. If not, upon recovery or rescaling you may have unpredictable redistribution of the partitions, thus unpredictable results.

        Type Parameters:
        K - The type of the keys in the broadcast state.
        V - The type of the values in the broadcast state.
        Parameters:
        stateDescriptor - The descriptor for this state, providing a name, a serializer for the keys and one for the values.
        Returns:
        The Broadcast State
        Throws:
        Exception
      • getListState

        <S> org.apache.flink.api.common.state.ListState<S> getListState​(ListStateDescriptor<S> stateDescriptor)
                                                                 throws Exception
        Creates (or restores) a list state. Each state is registered under a unique name. The provided serializer is used to de/serialize the state in case of checkpointing (snapshot/restore).

        Note the semantic differences between an operator list state and a keyed list state (see KeyedStateStore.getListState(ListStateDescriptor)). Under the context of operator state, the list is a collection of state items that are independent of each other and eligible for redistribution across operator instances in case of changed operator parallelism. In other words, these state items are the finest granularity at which non-keyed state can be redistributed, and should not be correlated with each other.

        The redistribution scheme of this list state upon operator rescaling is a round-robin pattern, such that the logical whole state (a concatenation of all the lists of state elements previously managed by each operator before the restore) is evenly divided into as many sublists as there are parallel operators.

        Type Parameters:
        S - The generic type of the state
        Parameters:
        stateDescriptor - The descriptor for this state, providing a name and serializer.
        Returns:
        A list for all state partitions.
        Throws:
        Exception
      • getUnionListState

        <S> org.apache.flink.api.common.state.ListState<S> getUnionListState​(ListStateDescriptor<S> stateDescriptor)
                                                                      throws Exception
        Creates (or restores) a list state. Each state is registered under a unique name. The provided serializer is used to de/serialize the state in case of checkpointing (snapshot/restore).

        Note the semantic differences between an operator list state and a keyed list state (see KeyedStateStore.getListState(ListStateDescriptor)). Under the context of operator state, the list is a collection of state items that are independent of each other and eligible for redistribution across operator instances in case of changed operator parallelism. In other words, these state items are the finest granularity at which non-keyed state can be redistributed, and should not be correlated with each other.

        The redistribution scheme of this list state upon operator rescaling is a broadcast pattern, such that the logical whole state (a concatenation of all the lists of state elements previously managed by each operator before the restore) is restored to all parallel operators so that each of them will get the union of all state items before the restore.

        Type Parameters:
        S - The generic type of the state
        Parameters:
        stateDescriptor - The descriptor for this state, providing a name and serializer.
        Returns:
        A list for all state partitions.
        Throws:
        Exception
      • getRegisteredStateNames

        Set<String> getRegisteredStateNames()
        Returns a set with the names of all currently registered states.
        Returns:
        set of names for all registered states.
      • getRegisteredBroadcastStateNames

        Set<String> getRegisteredBroadcastStateNames()
        Returns a set with the names of all currently registered broadcast states.
        Returns:
        set of names for all registered broadcast states.