Interface IStateStorage

All Superinterfaces:
AutoCloseable, Closeable
All Known Implementing Classes:
PaceMakerStateStorage, ZKStateStorage

public interface IStateStorage extends Closeable
StateStorage provides the API for the pluggable state store used by the Storm daemons. Data is stored in path/value format, and the store supports listing sub-paths at a given path. All data should be available across all nodes with eventual consistency.

IMPORTANT NOTE: Heartbeats have different api calls used to interact with them. The root path (/) may or may not be the same as the root path for the other api calls.

For example, performing these two calls: set_data("/path", data, acls); void set_worker_hb("/path", heartbeat, acls); may or may not cause a collision in "/path". Never use the same paths with the *_hb* methods as you do with the others.

  • Method Summary

    Modifier and Type
    Method
    Description
    void
    add_listener(org.apache.storm.shade.org.apache.curator.framework.state.ConnectionStateListener listener)
    Add a StateStorageListener to the connection.
    void
    Close the connection to the data store.
    create_sequential(String path, byte[] data, List<org.apache.storm.shade.org.apache.zookeeper.data.ACL> acls)
    Path will be appended with a monotonically increasing integer, a new node will be created there, and data will be put at that node.
    void
    Deletes the node at a given path, and any child nodes that may exist.
    void
    delete_node_blobstore(String path, String nimbusHostPortInfo)
    Allows us to delete the znodes within /storm/blobstore/key_name whose znodes start with the corresponding nimbusHostPortInfo.
    void
    Deletes the heartbeat at a given path, and any child nodes that may exist.
    get_children(String path, boolean watch)
    Get a list of paths of all the child nodes which exist immediately under path.
    byte[]
    get_data(String path, boolean watch)
    Get the data from the node at path
    get_data_with_version(String path, boolean watch)
    Get the data at the node along with its version.
    get_version(String path, boolean watch)
    Gets the 'version' of the node at a path.
    byte[]
    get_worker_hb(String path, boolean watch)
    Get the heartbeat from the node at path
    get_worker_hb_children(String path, boolean watch)
    Get a list of paths of all the child nodes which exist immediately under path.
    void
    mkdirs(String path, List<org.apache.storm.shade.org.apache.zookeeper.data.ACL> acls)
    Creates nodes for path and all its parents.
    boolean
    node_exists(String path, boolean watch)
    Check if a node exists and optionally set a watch on the path.
    Registers a callback function that gets called when CuratorEvents happen.
    void
    set_data(String path, byte[] data, List<org.apache.storm.shade.org.apache.zookeeper.data.ACL> acls)
    Set the value of the node at path to data.
    void
    set_ephemeral_node(String path, byte[] data, List<org.apache.storm.shade.org.apache.zookeeper.data.ACL> acls)
    Creates an ephemeral node at path.
    void
    set_worker_hb(String path, byte[] data, List<org.apache.storm.shade.org.apache.zookeeper.data.ACL> acls)
    Write a worker heartbeat at the path.
    void
    Force consistency on a path.
    void
    Unregisters a callback function that was registered with register(...).
  • Method Details

    • register

      String register(ZKStateChangedCallback callback)
      Registers a callback function that gets called when CuratorEvents happen.
      Parameters:
      callback - is a clojure IFn that accepts the type - translated to clojure keyword as in zookeeper - and the path: (callback type path)
      Returns:
      is an id that can be passed to unregister(...) to unregister the callback.
    • unregister

      void unregister(String id)
      Unregisters a callback function that was registered with register(...).
      Parameters:
      id - is the String id that was returned from register(...).
    • create_sequential

      String create_sequential(String path, byte[] data, List<org.apache.storm.shade.org.apache.zookeeper.data.ACL> acls)
      Path will be appended with a monotonically increasing integer, a new node will be created there, and data will be put at that node.
      Parameters:
      path - The path that the monotonically increasing integer suffix will be added to.
      data - The data that will be written at the suffixed path's node.
      acls - The acls to apply to the path. May be null.
      Returns:
      The path with the integer suffix appended.
    • mkdirs

      void mkdirs(String path, List<org.apache.storm.shade.org.apache.zookeeper.data.ACL> acls)
      Creates nodes for path and all its parents. Path elements are separated by a "/", as in *nix filesystem notation. Equivalent to mkdir -p in *nix.
      Parameters:
      path - The path to create, along with all its parents.
      acls - The acls to apply to the path. May be null.
    • delete_node

      void delete_node(String path)
      Deletes the node at a given path, and any child nodes that may exist.
      Parameters:
      path - The path to delete
    • set_ephemeral_node

      void set_ephemeral_node(String path, byte[] data, List<org.apache.storm.shade.org.apache.zookeeper.data.ACL> acls)
      Creates an ephemeral node at path. Ephemeral nodes are destroyed by the store when the client disconnects.
      Parameters:
      path - The path where a node will be created.
      data - The data to be written at the node.
      acls - The acls to apply to the path. May be null.
    • get_version

      Integer get_version(String path, boolean watch) throws Exception
      Gets the 'version' of the node at a path. Optionally sets a watch on that node. The version should increase whenever a write happens.
      Parameters:
      path - The path to get the version of.
      watch - Whether or not to set a watch on the path. Watched paths emit events which are consumed by functions registered with the register method. Very useful for catching updates to nodes.
      Returns:
      The integer version of this node.
      Throws:
      Exception
    • node_exists

      boolean node_exists(String path, boolean watch)
      Check if a node exists and optionally set a watch on the path.
      Parameters:
      path - The path to check for the existence of a node.
      watch - Whether or not to set a watch on the path. Watched paths emit events which are consumed by functions registered with the register method. Very useful for catching updates to nodes.
      Returns:
      Whether or not a node exists at path.
    • get_children

      List<String> get_children(String path, boolean watch)
      Get a list of paths of all the child nodes which exist immediately under path.
      Parameters:
      path - The path to look under
      watch - Whether or not to set a watch on the path. Watched paths emit events which are consumed by functions registered with the register method. Very useful for catching updates to nodes.
      Returns:
      list of string paths under path.
    • close

      void close()
      Close the connection to the data store.
      Specified by:
      close in interface AutoCloseable
      Specified by:
      close in interface Closeable
    • set_data

      void set_data(String path, byte[] data, List<org.apache.storm.shade.org.apache.zookeeper.data.ACL> acls)
      Set the value of the node at path to data.
      Parameters:
      path - The path whose node we want to set.
      data - The data to put in the node.
      acls - The acls to apply to the path. May be null.
    • get_data

      byte[] get_data(String path, boolean watch)
      Get the data from the node at path
      Parameters:
      path - The path to look under
      watch - Whether or not to set a watch on the path. Watched paths emit events which are consumed by functions registered with the register method. Very useful for catching updates to nodes.
      Returns:
      The data at the node.
    • get_data_with_version

      VersionedData<byte[]> get_data_with_version(String path, boolean watch)
      Get the data at the node along with its version. Data is returned in an Map with the keys data and version.
      Parameters:
      path - The path to look under
      watch - Whether or not to set a watch on the path. Watched paths emit events which are consumed by functions registered with the register method. Very useful for catching updates to nodes.
      Returns:
      the data with a version
    • set_worker_hb

      void set_worker_hb(String path, byte[] data, List<org.apache.storm.shade.org.apache.zookeeper.data.ACL> acls)
      Write a worker heartbeat at the path.
      Parameters:
      path - The path whose node we want to set.
      data - The data to put in the node.
      acls - The acls to apply to the path. May be null.
    • get_worker_hb

      byte[] get_worker_hb(String path, boolean watch)
      Get the heartbeat from the node at path
      Parameters:
      path - The path to look under
      watch - Whether or not to set a watch on the path. Watched paths emit events which are consumed by functions registered with the register method. Very useful for catching updates to nodes.
      Returns:
      The heartbeat at the node.
    • get_worker_hb_children

      List<String> get_worker_hb_children(String path, boolean watch)
      Get a list of paths of all the child nodes which exist immediately under path. This is similar to get_children, but must be used for any nodes
      Parameters:
      path - The path to look under
      watch - Whether or not to set a watch on the path. Watched paths emit events which are consumed by functions registered with the register method. Very useful for catching updates to nodes.
      Returns:
      list of string paths under path.
    • delete_worker_hb

      void delete_worker_hb(String path)
      Deletes the heartbeat at a given path, and any child nodes that may exist.
      Parameters:
      path - The path to delete.
    • add_listener

      void add_listener(org.apache.storm.shade.org.apache.curator.framework.state.ConnectionStateListener listener)
      Add a StateStorageListener to the connection.
      Parameters:
      listener - A StateStorageListener to handle changing cluster state events.
    • sync_path

      void sync_path(String path)
      Force consistency on a path. Any writes committed on the path before this call will be completely propagated when it returns.
      Parameters:
      path - The path to synchronize.
    • delete_node_blobstore

      void delete_node_blobstore(String path, String nimbusHostPortInfo)
      Allows us to delete the znodes within /storm/blobstore/key_name whose znodes start with the corresponding nimbusHostPortInfo.
      Parameters:
      path - /storm/blobstore/key_name
      nimbusHostPortInfo - Contains the host port information of a nimbus node.