public interface IStateStorage
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.
Modifier and Type | Method and Description |
---|---|
void |
add_listener(org.apache.curator.framework.state.ConnectionStateListener listener)
Add a StateStorageListener to the connection.
|
void |
close()
Close the connection to the data store.
|
String |
create_sequential(String path,
byte[] data,
List<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 |
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 |
delete_node(String path)
Deletes the node at a given path, and any child nodes that may exist.
|
void |
delete_worker_hb(String path)
Deletes the heartbeat at a given path, and any child nodes that may exist.
|
List<String> |
get_children(String path,
boolean watch)
Get a list of paths of all the child nodes which exist immediately under path.
|
VersionedData<byte[]> |
get_data_with_version(String path,
boolean watch)
Get the data at the node along with its version.
|
byte[] |
get_data(String path,
boolean watch)
Get the data from the node at path
|
Integer |
get_version(String path,
boolean watch)
Gets the ‘version’ of the node at a path.
|
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.
|
byte[] |
get_worker_hb(String path,
boolean watch)
Get the heartbeat from the node at path
|
void |
mkdirs(String path,
List<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.
|
String |
register(ZKStateChangedCallback callback)
Registers a callback function that gets called when CuratorEvents happen.
|
void |
set_data(String path,
byte[] data,
List<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.zookeeper.data.ACL> acls)
Creates an ephemeral node at path.
|
void |
set_worker_hb(String path,
byte[] data,
List<org.apache.zookeeper.data.ACL> acls)
Write a worker heartbeat at the path.
|
void |
sync_path(String path)
Force consistency on a path.
|
void |
unregister(String id)
Unregisters a callback function that was registered with register(…).
|
String register(ZKStateChangedCallback callback)
Registers a callback function that gets called when CuratorEvents happen.
callback
- is a clojure IFn that accepts the type - translated to clojure keyword as in zookeeper - and the path: (callback type path)void unregister(String id)
Unregisters a callback function that was registered with register(…).
id
- is the String id that was returned from register(…).String create_sequential(String path, byte[] data, List<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.
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.void mkdirs(String path, List<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.
path
- The path to create, along with all its parents.acls
- The acls to apply to the path. May be null.void delete_node(String path)
Deletes the node at a given path, and any child nodes that may exist.
path
- The path to deletevoid set_ephemeral_node(String path, byte[] data, List<org.apache.zookeeper.data.ACL> acls)
Creates an ephemeral node at path. Ephemeral nodes are destroyed by the store when the client disconnects.
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.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.
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.Exception
boolean node_exists(String path, boolean watch)
Check if a node exists and optionally set a watch on the path.
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.List<String> get_children(String path, boolean watch)
Get a list of paths of all the child nodes which exist immediately under path.
path
- The path to look underwatch
- 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.void close()
Close the connection to the data store.
void set_data(String path, byte[] data, List<org.apache.zookeeper.data.ACL> acls)
Set the value of the node at path to data.
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.byte[] get_data(String path, boolean watch)
Get the data from the node at path
path
- The path to look underwatch
- 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.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.
path
- The path to look underwatch
- 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.void set_worker_hb(String path, byte[] data, List<org.apache.zookeeper.data.ACL> acls)
Write a worker heartbeat at the path.
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.byte[] get_worker_hb(String path, boolean watch)
Get the heartbeat from the node at path
path
- The path to look underwatch
- 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.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
path
- The path to look underwatch
- 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.void delete_worker_hb(String path)
Deletes the heartbeat at a given path, and any child nodes that may exist.
path
- The path to delete.void add_listener(org.apache.curator.framework.state.ConnectionStateListener listener)
Add a StateStorageListener to the connection.
listener
- A StateStorageListener to handle changing cluster state events.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.
path
- The path to synchronize.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
path
- /storm/blobstore/key_namenimbusHostPortInfo
- Contains the host port information of a nimbus node.Copyright © 2019 The Apache Software Foundation. All Rights Reserved.