Interface TaskChangelogRegistry


@Internal public interface TaskChangelogRegistry
TM-side registry of StateObjects, each representing one or more changelog segments. Changelog segments are uploaded by StateChangelogWriters of a StateChangelogStorage.

Initially, when starting the tracking, the ownership of a changelog segments is not clear, and it is assumed that JM might be the owner. The refCount of the StateObject refers to the number of changelog segments contains in the StateObject. release(StreamStateHandle) should be called when every changelog segment become not used, and it will count down the refCount by one. Once the refCount reaches zero, JM can not become an owner anymore, the state is discarded.

However, if at any point it becomes known that JM is the owner, tracking is stopped and the state will not be discarded.

It is the client responsibility to call release(StreamStateHandle) when every corresponding changelog segment becomes not used.

  • Field Summary

    Fields
    Modifier and Type
    Field
    Description
     
  • Method Summary

    Modifier and Type
    Method
    Description
    defaultChangelogRegistry(int numAsyncDiscardThreads)
     
     
    void
    release(org.apache.flink.runtime.state.StreamStateHandle handle)
    Decrease the reference count of the state by one, e.g. if it was pre-emptively uploaded and materialized.
    void
    startTracking(org.apache.flink.runtime.state.StreamStateHandle handle, long refCount)
    Start tracking the state uploaded.
    void
    stopTracking(org.apache.flink.runtime.state.StreamStateHandle handle)
    Stop tracking the state, so that it's not tracked (some other component is doing that).
  • Field Details

  • Method Details

    • startTracking

      void startTracking(org.apache.flink.runtime.state.StreamStateHandle handle, long refCount)
      Start tracking the state uploaded. The refCount is the number of StateChangeSets.
    • stopTracking

      void stopTracking(org.apache.flink.runtime.state.StreamStateHandle handle)
      Stop tracking the state, so that it's not tracked (some other component is doing that).
    • release

      void release(org.apache.flink.runtime.state.StreamStateHandle handle)
      Decrease the reference count of the state by one, e.g. if it was pre-emptively uploaded and materialized. Once the reference count reaches zero, it is discarded (unless it was unregistered earlier).
    • defaultChangelogRegistry

      static TaskChangelogRegistry defaultChangelogRegistry(int numAsyncDiscardThreads)
    • defaultChangelogRegistry

      @VisibleForTesting static TaskChangelogRegistry defaultChangelogRegistry(Executor executor)