Class Config
- All Implemented Interfaces:
Serializable
,Cloneable
,Map<String,
Object>
This class also provides constants for all the configurations possible on a Storm cluster and Storm topology. Each constant is paired with an annotation that defines the validity criterion of the corresponding field. Default values for these configs can be found in defaults.yaml.
Note that you may put other configurations in any of the configs. Storm will ignore anything it doesn't recognize, but your topologies are free to make use of them by reading them in the prepare method of Bolts or the open method of Spouts.
- See Also:
-
Nested Class Summary
Nested classes/interfaces inherited from class java.util.AbstractMap
AbstractMap.SimpleEntry<K extends Object,
V extends Object>, AbstractMap.SimpleImmutableEntry<K extends Object, V extends Object> -
Field Summary
Modifier and TypeFieldDescriptionstatic final String
Enable the blobstore cleaner.static final String
What directory to use for the blobstore.static final String
Deprecated.static final String
Deprecated, for removal: This API element is subject to removal in a future version.static final String
What blobstore implementation the storm client should use.static final String
On some systems (windows for example) symlinks require special privileges that not everyone wants to grant a headless user.static final String
The Access Control List for the DRPC Authorizer.static final String
File name of the DRPC Authorizer ACL.static final String
Whether the DRPCSimpleAclAuthorizer should deny requests for operations involving functions that have no explicit ACL entry.static final String
This port on Storm DRPC is used by DRPC topologies to receive function invocations and send results back.static final String
DRPC invocations thrift server worker threads.static final String
The DRPC invocations transport plug-in for Thrift client/server communication.static final String
The maximum buffer size thrift should use when reading messages for DRPC.static final String
This port is used by Storm DRPC for receiving DPRC requests from clients.static final String
DRPC thrift server queue size.static final String
List of DRPC servers so that the DRPCSpout knows who to talk to.static final String
The DRPC transport plug-in for Thrift client/server communication.static final String
DRPC thrift server worker threads.static final String
How often executor metrics should report to master, used for RPC heartbeat mode.static final String
A list of users that are cluster admins and can run any command.static final String
A list of groups that are cluster admins and can run any command.static final String
A list of plugins that nimbus should load during submit topology to populate credentials on user's behalf.static final String
A list of credential renewers that nimbus should load.static final String
A list of users that nimbus runs as and should be authorized to interact with the supervisor as nimbus would.static final String
A list of groups , users belong to these groups are the only ones allowed to run user operation on storm cluster.static final String
Impersonation user ACL config entries.static final String
Nimbus assignments backend for storing local assignments.static final String
Nimbus thrift server queue size, default is 100000.static final String
A whitelist of the RAS scheduler strategies allowed by nimbus.static final String
List of seed nimbus hosts to use for leader nimbus discovery.static final String
A list of users that run the supervisors and should be authorized to interact with nimbus as a supervisor would.static final String
The maximum buffer size thrift should use when reading messages.static final String
Which port the Thrift interface of Nimbus should run on.static final String
The number of threads that should be used by the nimbus thrift server.static final String
The Nimbus transport plug-in for Thrift client/server communication.static final String
A list of users that are the only ones allowed to run user operation on storm cluster.static final String
The number of Buckets.static final String
This should be one of "DIGEST", "KERBEROS", or "NONE" Determines the mode of authentication the pacemaker server and client use.static final String
The maximum number of threads that should be used by the Pacemaker client.static final String
The port Pacemaker should run on.static final String
The list of servers that Pacemaker is running on.static final String
Pacemaker Thrift Max Message Size (bytes).static final String
For secure mode we would want to turn on this config By default this is turned off assuming the default is insecure.static final String
What chunk size to use for storm client to upload dependency jars.static final String
What buffer size to use for the blobstore uploads.static final String
Set replication factor for a blob in HDFS Blobstore Implementation.static final String
root directory of the storm cgroup hierarchy.static final String
The mode this Storm cluster is running in.static final String
The ClusterState factory that worker will use to create a ClusterState to store state in.static final String
Specify the duration unit in TimeUnit for daemon metrics reporter plugin.static final String
Specify the Locale for daemon metrics reporter plugin.static final String
Specify the rate unit in TimeUnit for daemon metrics reporter plugin.static final String
The user as which the nimbus client should be acquired to perform the operation.static final String
Max no.of seconds group mapping service will cache user groupsstatic final String
Initialization parameters for the group mapping service plugin.static final String
The plugin that will provide user groups service.static final String
The keytab for nimbus/supervisor to use to access secure hdfs.static final String
The principal for nimbus/supervisor to use to access secure hdfs.static final String
The id assigned to a running topology.static final String
A directory on the local filesystem used by Storm for any local filesystem usage it needs.static final String
The hostname the supervisors/workers should report to nimbus.static final String
Whether or not to use ZeroMQ for messaging in local mode.static final String
Netty based messaging: Is authentication required for Netty messaging from client worker process to server worker process.static final String
Netty based messaging: The netty write buffer high watermark in bytes.static final String
Netty based messaging: The netty write buffer low watermark in bytes.static final String
Netty based messaging: The buffer size for send/recv buffer.static final String
Netty based messaging: The # of worker threads for the client.static final String
Netty based messaging: The max # of milliseconds that a peer will wait.static final String
Netty based messaging: The min # of milliseconds that a peer will wait.static final String
Netty based messaging: The # of worker threads for the server.static final String
Netty based messaging: Sets the backlog value to specify when the channel binds to a local address.static final String
The transporter for communication among Storm tasks.static final String
The delegate for serializing metadata, should be used for serialized objects stored in zookeeper and on disk.static final String
Deprecated, for removal: This API element is subject to removal in a future version.static final String
If the Netty messaging layer is busy, the Netty client will try to batch message as more as possible up to the size of STORM_NETTY_MESSAGE_BATCH_SIZE bytes.static final String
What Network Topography detection classes should we use.static final String
The starting interval between exponential backoff retries of a Nimbus operation.static final String
The ceiling of the interval between retries of a client connect to Nimbus operation.static final String
The number of times to retry a Nimbus operation.static String
The root of cgroup for oci to use.static final String
The plugin that will convert a principal to a local user.static final String
How long before a Thrift Client socket hangs before timeout and restart the socket.static final String
The default transport plug-in for Thrift client/server communication.static final String
A list of system metrics reporters that will get added to each topology.static final String
FQCN of a class that implementsISubmitterHook
@see ISubmitterHook for details.static final String
The workers-artifacts directory (where we place all workers' logs), can be either absolute or relative.static final String
A string representing the payload for cluster Zookeeper authentication.static final String
The cluster Zookeeper authentication scheme to use, e.g.static final String
The connection timeout for clients to ZooKeeper.static final String
The ACL of the drpc user in zookeeper so the drpc servers can verify worker tokens.static final String
The port Storm will use to connect to each of the ZooKeeper servers.static final String
The interval between retries of a Zookeeper operation.static final String
The ceiling of the interval between retries of a Zookeeper operation.static final String
The number of times to retry a Zookeeper operation.static final String
The root location at which Storm stores data in ZooKeeper.static final String
A list of hosts of ZooKeeper servers used to manage the cluster.static final String
The session timeout for clients to ZooKeeper.static final String
This is part of a temporary workaround to a ZK bug, it is the 'scheme:acl' for the user Nimbus and Supervisors use to authenticate with ZK.static final String
A string representing the payload for topology Zookeeper authentication.static final String
The topology Zookeeper authentication scheme to use, e.g.static final String
The total amount of CPU resources a supervisor is allowed to give to its workers.static final String
The total amount of memory (in MiB) a supervisor is allowed to give to its workers.static final String
static final String
A map of resources the Supervisor has e.g {"cpu.pcore.percent" : 200.0.static final String
Should the supervior try to run the worker as the lauching user or not.static final String
static final String
static final String
How long before a supervisor Thrift Client socket hangs before timeout and restart the socket.static final String
static final String
static final String
The version of storm to assume a topology should run as if not version is given by the client when submitting the topology.static final String
max timeout for supervisor reported heartbeats when master gains leadership.static final String
Full path to the worker-laucher executable that will be used to lauch workers when SUPERVISOR_RUN_WORKER_AS_USER is set to true.static final String
How many seconds to allow for graceful worker shutdown when killing workers before resorting to force kill.static final String
How long a worker can go without heartbeating before the supervisor tries to restart the worker process.static final String
Map a version of storm to a worker classpath that can be used to run it.static final String
Map a version of storm to a worker's logwriter class.static final String
Map a version of storm to a worker's main class.static final String
How often a task should sync credentials, worst case.static final String
How often a task should heartbeat its status to the Pacamker.static final String
How often a task should sync its connections with other tasks (if a task is reassigned, the other tasks sending messages to it need to refresh their connections).static final String
The config indicates the percentage of cpu for a core an instance(executor) of an acker will use.static final String
How many executors to spawn for ackers.static final String
The maximum amount of memory an instance of an acker will take off heap.static final String
The maximum amount of memory an instance of an acker will take on heap.static final String
A list of IAutoCredentials that the topology should load and use.static final String
A list of task hooks that are automatically added to every spout and bolt in the topology.static final String
How often a worker should check and notify upstream workers about its tasks that are no longer experiencing BP and able to receive new messages.static final String
Deprecated, for removal: This API element is subject to removal in a future version.static final String
Configures park time if using WaitStrategyPark for BackPressure.static final String
Configures steps used to determine progression to the next level of wait .. if using WaitStrategyProgressive for BackPressure.static final String
Configures steps used to determine progression to the next level of wait .. if using WaitStrategyProgressive for BackPressure.static final String
Configures sleep time if using WaitStrategyProgressive for BackPressure.static final String
A class that implements a wait strategy for an upstream component (spout/bolt) trying to write to a downstream component whose recv queue is fullstatic final String
How often to send flush tuple to the executors for flushing out batched events.static final String
A map with blobstore keys mapped to each filename the worker will have access to in the launch directory to the blob by local file name, uncompress flag, and if the worker should restart when the blob is updated. localname, workerRestart, and uncompress are optional.static final String
Configures park time for WaitStrategyPark.static final String
Configures number of iterations to spend in level 1 of WaitStrategyProgressive, before progressing to level 2.static final String
Configures number of iterations to spend in level 2 of WaitStrategyProgressive, before progressing to level 3.static final String
Configures sleep time for WaitStrategyProgressive.static final String
Selects the Bolt's Wait Strategy to use when there are no incoming msgs.static final String
Bolt-specific configuration for windowed bolts to specify the name of the stream on which late tuples are going to be emitted.static final String
static final String
static final String
static final String
Bolt-specific configuration for windowed bolts to specify the maximum time lag of the tuple timestamp in milliseconds.static final String
static final String
static final String
static final String
The time period that builtin metrics data in bucketed into.static final String
Topology-specific classpath for the worker child process.static final String
Topology-specific classpath for the worker child process.static final String
The config indicates the percentage of cpu for a core an instance(executor) of a component will use.static final String
A map of resources used by each component e.g {"cpu.pcore.percent" : 200.0.static final String
The maximum amount of memory an instance of a spout/bolt will take off heap.static final String
The maximum amount of memory an instance of a spout/bolt will take on heap.static final String
When set to true, Storm will log every message that's emitted.static final String
Disable load aware grouping support.static final String
True if Storm should timeout messages or not.static final String
This config allows a topology to enable/disable reporting of __send-iconnection metrics.static final String
This config allows a topology to report metrics data points from the V2 metrics API through the metrics tick.static final String
Topology-specific environment variables for the worker child process.static final String
The interval in seconds to use for determining whether to throttle error reported to Zookeeper.static final String
A list of classes implementing IEventLogger (See storm.yaml.example for exact config format).static final String
How many executors to spawn for event logger.static final String
If number of items in task's overflowQ exceeds this, new messages coming from other workers to this task will be dropped This prevents OutOfMemoryException that can occur in rare scenarios in the presence of BackPressure.static final String
The size of the receive queue for each executor.static final String
Whether or not to use Java serialization in a topology.static final String
A list of groups that are allowed to interact with the topology.static final String
The maximum number of machines that should be used by this topology.static final String
A list of classes that customize storm's kryo instance during start-up.static final String
Class that specifies how to create a Kryo instance for serialization.static final String
A list of serialization registrations for Kryo ( https://github.com/EsotericSoftware/kryo ), the underlying serialization framework for Storm.static final String
This signifies the load congestion among target tasks in scope.static final String
This signifies the load congestion among target tasks in scope.static final String
Log file the user can use to configure Log4j2.static final String
Topology central logging sensitivity to determine who has access to logs in central logging system.static final String
See doc forTOPOLOGY_ERROR_THROTTLE_INTERVAL_SECS
.static final String
Maximum wait time for the nimbus host replication to achieve the nimbus.min.replication.count.static final String
The maximum number of tuples that can be pending on a spout task at any given time.static final String
The maximum parallelism allowed for a component in this topology.static final String
The maximum amount of time given to the topology to fully process a message emitted by a spout.static final String
The config indicates the percentage of cpu for a core an instance(executor) of a metrics consumer will use.static final String
A list of classes implementing IMetricsConsumer (See storm.yaml.example for exact config format).static final String
The maximum amount of memory an instance of a metrics consumer will take off heap.static final String
The maximum amount of memory an instance of a metrics consumer will take on heap.static final String
Configure the topology metrics reporters to be used on workers.static final String
Minimum number of nimbus hosts where the code must be replicated before leader nimbus is allowed to perform topology activation tasks like setting up heartbeats/assignments and marking the topology as active. default is 0.static final String
The fully qualified name of aShellLogHandler
to handle output from non-JVM processes e.g.static final String
The serializer for communication between shell components and non-JVM processes.static final String
Name of the topology.static String
Specify the oci image to use.static final String
Sets the priority for a topology.static final String
The number of tuples to batch before sending to the destination executor.static final String
How many ackers to put in when launching a new worker until we run out of ackers.static final String
The maximum number of states that will be searched looking for a solution in resource aware strategies, e.g.static final String
The maximum number of seconds to spend scheduling a topology using resource aware strategies, e.g.static final String
Declare scheduling constraints for a topology used by the constraint solver strategy.static final String
Whether to limit each worker to one component.static final String
static final String
If set to true, unassigned executors will be sorted by topological order with network proximity needs before being scheduled.static final String
A list of readonly groups that are allowed to interact with the topology.static final String
A list of readonly users that are allowed to interact with the topology.static final String
A list of host names that this topology would prefer to be scheduled on (no guarantee is given though).static final String
The strategy to use when scheduling a topology with Resource Aware Scheduler.static final String
A list of host names that this topology would prefer to NOT be scheduled on (no guarantee is given though).static final String
Enable tracking of network message byte counts per source-destination task.static final String
Max pending tuples in one ShellBolt.static final String
Whether or not Storm should skip the loading of kryo registrations for which it does not know the class or have the serializer implementation.static final String
The amount of milliseconds the SleepEmptyEmitStrategy should sleep for.static final String
Check recvQ after every N invocations of Spout's nextTuple() [when ACKing is disabled].static final String
Configures park time for WaitStrategyPark for spout.static final String
Configures number of iterations to spend in level 1 of WaitStrategyProgressive, before progressing to level 2.static final String
Configures number of iterations to spend in level 2 of WaitStrategyProgressive, before progressing to level 3.static final String
Configures sleep time for WaitStrategyProgressive.static final String
A class that implements a wait strategy for spout.static final String
Deprecated.static final String
Topology configuration to specify the checkpoint interval (in millis) at which the topology state is saved whenIStatefulBolt
bolts are involved.static final String
List of classes to register during state serialization.static final String
The class name of theStateProvider
implementation.static final String
The configuration specific to theStateProvider
implementation.static final String
The maximum amount of time a component gives a source of state to synchronize before it requests synchronization again.static final String
The percentage of tuples to sample to produce stats for a task.static final String
The principal who submitted a topology.static final String
The local user name of the user who submitted a topology.static final String
How long a subprocess can go without heartbeating before the ShellSpout/ShellBolt tries to suicide itself.static final String
How many instances to create for a spout/bolt.static final String
Try to serialize all tuples, even for local transfers.static final String
How often a tick tuple from the "__system" component and "__tick" stream should be sent to tasks.static final String
This config is available for TransactionalSpouts, and contains the id ( a String) for the transactional topology.static final String
The size of the transfer queue for each worker.static final String
The size of the transfer queue for each worker.static final String
How often a batch can be emitted in a Trident topology.static final String
Maximum number of tuples that can be stored inmemory cache in windowing operators for fast access without fetching them from store.static final String
The serializer class for ListDelegate (tuple payload).static final String
A list of users that are allowed to interact with the topology.static final String
Topology configuration to specify the V2 metrics tick interval in seconds.static final String
User defined version of this topology.static final String
Topology-specific options for the worker child process.static final String
Topology-specific options GC for the worker child process.static final String
Topology-specific options for the logwriter process of a worker.static final String
A per topology config that specifies the maximum amount of memory a worker can use for that specific topology.static final String
A map of metric name to class name implementing IMetric that will be created once per worker JVM.static final String
The size of the shared thread pool for worker tasks to make use of.static final String
Topology configurable worker heartbeat timeout before the supervisor tries to restart the worker process.static final String
How many processes should be spawned around the cluster to execute this topology.static final String
The port to use to connect to the transactional zookeeper servers.static final String
The root directory in ZooKeeper for metadata about TransactionalSpouts.static final String
The list of zookeeper servers in which to keep the transactional state.static final String
Interval to check for the worker to check for updated blobs and refresh worker state accordingly.static final String
The jvm opts provided to workers launched by this supervisor.static final String
The jvm opts provided to workers launched by this supervisor for GC.static final String
The default heap memory size in MB per worker, used in the jvm -Xmx opts for launching the worker.static final String
How often this worker should heartbeat to the supervisor.static final String
How often a worker should check dynamic log level timeouts for expiration.static final String
Enforce maximum onTOPOLOGY_WORKER_TIMEOUT_SECS
.static final String
A map of metric name to class name implementing IMetric that will be created once per worker JVM. -
Constructor Summary
-
Method Summary
Modifier and TypeMethodDescriptionstatic String
getBlobstoreHDFSPrincipal
(Map conf) Deprecated.static String
getHdfsKeytab
(Map<String, Object> conf) Get the hdfs keytab.static String
getHdfsPrincipal
(Map<String, Object> conf) Get the hostname substituted hdfs principal.void
registerDecorator
(Class<? extends IKryoDecorator> klass) static void
registerDecorator
(Map<String, Object> conf, Class<? extends IKryoDecorator> klass) void
registerEventLogger
(Class<? extends IEventLogger> klass) void
registerEventLogger
(Class<? extends IEventLogger> klass, Map<String, Object> argument) static void
registerEventLogger
(Map<String, Object> conf, Class<? extends IEventLogger> klass) static void
registerEventLogger
(Map<String, Object> conf, Class<? extends IEventLogger> klass, Map<String, Object> argument) void
registerMetricsConsumer
(Class klass) void
registerMetricsConsumer
(Class klass, long parallelismHint) void
registerMetricsConsumer
(Class klass, Object argument, long parallelismHint) static void
registerMetricsConsumer
(Map<String, Object> conf, Class klass) static void
registerMetricsConsumer
(Map<String, Object> conf, Class klass, long parallelismHint) static void
registerMetricsConsumer
(Map<String, Object> conf, Class klass, Object argument, long parallelismHint) void
registerSerialization
(Class klass) void
registerSerialization
(Class klass, Class<? extends com.esotericsoftware.kryo.Serializer> serializerClass) static void
registerSerialization
(Map<String, Object> conf, Class klass) static void
registerSerialization
(Map<String, Object> conf, Class klass, Class<? extends com.esotericsoftware.kryo.Serializer> serializerClass) void
setClasspath
(String cp) static void
setClasspath
(Map<String, Object> conf, String cp) void
setDebug
(boolean isOn) static void
void
setEnvironment
(Map<String, Object> env) static void
void
setFallBackOnJavaSerialization
(boolean fallback) static void
setFallBackOnJavaSerialization
(Map<String, Object> conf, boolean fallback) void
setKryoFactory
(Class<? extends IKryoFactory> klass) static void
setKryoFactory
(Map<String, Object> conf, Class<? extends IKryoFactory> klass) void
setMaxSpoutPending
(int max) static void
setMaxSpoutPending
(Map<String, Object> conf, int max) void
setMaxTaskParallelism
(int max) static void
setMaxTaskParallelism
(Map<String, Object> conf, int max) void
setMessageTimeoutSecs
(int secs) static void
setMessageTimeoutSecs
(Map<String, Object> conf, int secs) void
setNumAckers
(int numExecutors) static void
setNumAckers
(Map<String, Object> conf, int numExecutors) void
setNumEventLoggers
(int numExecutors) static void
setNumEventLoggers
(Map<String, Object> conf, int numExecutors) void
setNumWorkers
(int workers) static void
setNumWorkers
(Map<String, Object> conf, int workers) void
setSkipMissingKryoRegistrations
(boolean skip) static void
setSkipMissingKryoRegistrations
(Map<String, Object> conf, boolean skip) void
setStatsSampleRate
(double rate) static void
setStatsSampleRate
(Map<String, Object> conf, double rate) void
setTopologyComponentWorkerConstraints
(String component1, String component2) Declares executors of component1 cannot be on the same worker as executors of component2.void
setTopologyConstraintsMaxStateSearch
(int numStates) Sets the maximum number of states that will be searched in the constraint solver strategy.void
setTopologyPriority
(int priority) Set the priority for a topology.void
setTopologyStrategy
(String strategy) void
setTopologyVersion
(String version) static void
setTopologyVersion
(Map<String, Object> conf, String version) void
Set the max heap size allow per worker for this topology.Methods inherited from class java.util.HashMap
clear, clone, compute, computeIfAbsent, computeIfPresent, containsKey, containsValue, entrySet, forEach, get, getOrDefault, isEmpty, keySet, merge, put, putAll, putIfAbsent, remove, remove, replace, replace, replaceAll, size, values
Methods inherited from class java.util.AbstractMap
equals, hashCode, toString
-
Field Details
-
TOPOLOGY_TUPLE_SERIALIZER
The serializer class for ListDelegate (tuple payload). The default serializer will be ListDelegateSerializer- See Also:
-
TOPOLOGY_DISABLE_LOADAWARE_MESSAGING
Disable load aware grouping support.- See Also:
-
TOPOLOGY_LOCALITYAWARE_HIGHER_BOUND
This signifies the load congestion among target tasks in scope. Currently it's only used in LoadAwareShuffleGrouping. When the average load is higher than the higher bound, the executor should choose target tasks in a higher scope, The scopes and their orders are: EVERYTHING > RACK_LOCAL > HOST_LOCAL > WORKER_LOCAL- See Also:
-
TOPOLOGY_LOCALITYAWARE_LOWER_BOUND
This signifies the load congestion among target tasks in scope. Currently it's only used in LoadAwareShuffleGrouping. When the average load is lower than the lower bound, the executor should choose target tasks in a lower scope. The scopes and their orders are: EVERYTHING > RACK_LOCAL > HOST_LOCAL > WORKER_LOCAL- See Also:
-
TOPOLOGY_TESTING_ALWAYS_TRY_SERIALIZE
Try to serialize all tuples, even for local transfers. This should only be used for testing, as a sanity check that all of your tuples are setup properly.- See Also:
-
TOPOLOGY_BLOBSTORE_MAP
A map with blobstore keys mapped to each filename the worker will have access to in the launch directory to the blob by local file name, uncompress flag, and if the worker should restart when the blob is updated. localname, workerRestart, and uncompress are optional. If localname is not specified the name of the key is used instead. Each topologywill have different map of blobs. Example: topology.blobstore.map: {"blobstorekey" : {"localname": "myblob", "uncompress": false}, "blobstorearchivekey" : {"localname": "myarchive", "uncompress": true, "workerRestart": true}}- See Also:
-
WORKER_LOG_LEVEL_RESET_POLL_SECS
How often a worker should check dynamic log level timeouts for expiration. For expired logger settings, the clean up polling task will reset the log levels to the original levels (detected at startup), and will clean up the timeout map- See Also:
-
TASK_CREDENTIALS_POLL_SECS
How often a task should sync credentials, worst case.- See Also:
-
TOPOLOGY_BACKPRESSURE_ENABLE
Deprecated, for removal: This API element is subject to removal in a future version.Whether to enable backpressure in for a certain topology. Note: Retained for enabling transition from 1.x. Will be removed soon.- See Also:
-
TOPOLOGY_USERS
A list of users that are allowed to interact with the topology. To use this set nimbus.authorizer to org.apache.storm.security.auth.authorizer.SimpleACLAuthorizer- See Also:
-
TOPOLOGY_GROUPS
A list of groups that are allowed to interact with the topology. To use this set nimbus.authorizer to org.apache.storm.security.auth.authorizer.SimpleACLAuthorizer- See Also:
-
TOPOLOGY_READONLY_USERS
A list of readonly users that are allowed to interact with the topology. To use this set nimbus.authorizer to org.apache.storm.security.auth.authorizer.SimpleACLAuthorizer- See Also:
-
TOPOLOGY_READONLY_GROUPS
A list of readonly groups that are allowed to interact with the topology. To use this set nimbus.authorizer to org.apache.storm.security.auth.authorizer.SimpleACLAuthorizer- See Also:
-
TOPOLOGY_ENABLE_MESSAGE_TIMEOUTS
True if Storm should timeout messages or not. Defaults to true. This is meant to be used in unit tests to prevent tuples from being accidentally timed out during the test.- See Also:
-
TOPOLOGY_DEBUG
When set to true, Storm will log every message that's emitted.- See Also:
-
TOPOLOGY_VERSION
User defined version of this topology.- See Also:
-
TOPOLOGY_MULTILANG_LOG_HANDLER
The fully qualified name of aShellLogHandler
to handle output from non-JVM processes e.g. "com.mycompany.CustomShellLogHandler". If not provided, org.apache.storm.utils.DefaultLogHandler will be used.- See Also:
-
TOPOLOGY_MULTILANG_SERIALIZER
The serializer for communication between shell components and non-JVM processes.- See Also:
-
TOPOLOGY_WORKERS
How many processes should be spawned around the cluster to execute this topology. Each process will execute some number of tasks as threads within them. This parameter should be used in conjunction with the parallelism hints on each component in the topology to tune the performance of a topology. The number of workers will be dynamically calculated when the Resource Aware scheduler is used, in which case this parameter will not be honored.- See Also:
-
TOPOLOGY_TASKS
How many instances to create for a spout/bolt. A task runs on a thread with zero or more other tasks for the same spout/bolt. The number of tasks for a spout/bolt is always the same throughout the lifetime of a topology, but the number of executors (threads) for a spout/bolt can change over time. This allows a topology to scale to more or less resources without redeploying the topology or violating the constraints of Storm (such as a fields grouping guaranteeing that the same value goes to the same task).- See Also:
-
TOPOLOGY_COMPONENT_RESOURCES_MAP
A map of resources used by each component e.g {"cpu.pcore.percent" : 200.0. "onheap.memory.mb": 256.0, "gpu.count" : 2 }- See Also:
-
TOPOLOGY_COMPONENT_RESOURCES_ONHEAP_MEMORY_MB
The maximum amount of memory an instance of a spout/bolt will take on heap. This enables the scheduler to allocate slots on machines with enough available memory. A default value will be set for this config if user does not override- See Also:
-
TOPOLOGY_COMPONENT_RESOURCES_OFFHEAP_MEMORY_MB
The maximum amount of memory an instance of a spout/bolt will take off heap. This enables the scheduler to allocate slots on machines with enough available memory. A default value will be set for this config if user does not override- See Also:
-
TOPOLOGY_COMPONENT_CPU_PCORE_PERCENT
The config indicates the percentage of cpu for a core an instance(executor) of a component will use. Assuming the a core value to be 100, a value of 10 indicates 10% of the core. The P in PCORE represents the term "physical". A default value will be set for this config if user does not override- See Also:
-
TOPOLOGY_ACKER_RESOURCES_ONHEAP_MEMORY_MB
The maximum amount of memory an instance of an acker will take on heap. This enables the scheduler to allocate slots on machines with enough available memory. A default value will be set for this config if user does not override- See Also:
-
TOPOLOGY_ACKER_RESOURCES_OFFHEAP_MEMORY_MB
The maximum amount of memory an instance of an acker will take off heap. This enables the scheduler to allocate slots on machines with enough available memory. A default value will be set for this config if user does not override- See Also:
-
TOPOLOGY_ACKER_CPU_PCORE_PERCENT
The config indicates the percentage of cpu for a core an instance(executor) of an acker will use. Assuming the a core value to be 100, a value of 10 indicates 10% of the core. The P in PCORE represents the term "physical". A default value will be set for this config if user does not override- See Also:
-
TOPOLOGY_METRICS_CONSUMER_RESOURCES_ONHEAP_MEMORY_MB
The maximum amount of memory an instance of a metrics consumer will take on heap. This enables the scheduler to allocate slots on machines with enough available memory. A default value will be set for this config if user does not override- See Also:
-
TOPOLOGY_METRICS_CONSUMER_RESOURCES_OFFHEAP_MEMORY_MB
The maximum amount of memory an instance of a metrics consumer will take off heap. This enables the scheduler to allocate slots on machines with enough available memory. A default value will be set for this config if user does not override- See Also:
-
TOPOLOGY_METRICS_CONSUMER_CPU_PCORE_PERCENT
The config indicates the percentage of cpu for a core an instance(executor) of a metrics consumer will use. Assuming the a core value to be 100, a value of 10 indicates 10% of the core. The P in PCORE represents the term "physical". A default value will be set for this config if user does not override- See Also:
-
TOPOLOGY_ENABLE_V2_METRICS_TICK
This config allows a topology to report metrics data points from the V2 metrics API through the metrics tick.- See Also:
-
TOPOLOGY_V2_METRICS_TICK_INTERVAL_SECONDS
Topology configuration to specify the V2 metrics tick interval in seconds.- See Also:
-
TOPOLOGY_ENABLE_SEND_ICONNECTION_METRICS
This config allows a topology to enable/disable reporting of __send-iconnection metrics.- See Also:
-
TOPOLOGY_STATE_PROVIDER
The class name of theStateProvider
implementation. If not specified defaults toInMemoryKeyValueStateProvider
. This can be overridden at the component level.- See Also:
-
TOPOLOGY_STATE_PROVIDER_CONFIG
The configuration specific to theStateProvider
implementation. This can be overridden at the component level. The value and the interpretation of this config is based on the state provider implementation. For e.g. this could be just a config file name which contains the config for the state provider implementation.- See Also:
-
TOPOLOGY_STATE_CHECKPOINT_INTERVAL
Topology configuration to specify the checkpoint interval (in millis) at which the topology state is saved whenIStatefulBolt
bolts are involved.- See Also:
-
TOPOLOGY_WORKER_MAX_HEAP_SIZE_MB
A per topology config that specifies the maximum amount of memory a worker can use for that specific topology.- See Also:
-
TOPOLOGY_SCHEDULER_STRATEGY
The strategy to use when scheduling a topology with Resource Aware Scheduler.- See Also:
-
TOPOLOGY_RAS_ORDER_EXECUTORS_BY_PROXIMITY_NEEDS
If set to true, unassigned executors will be sorted by topological order with network proximity needs before being scheduled. This is a best-effort to split the topology to slices and allocate executors in each slice to closest physical location as possible.- See Also:
-
TOPOLOGY_RAS_CONSTRAINTS
Declare scheduling constraints for a topology used by the constraint solver strategy. The format can be either old style (validated by ListOfListOfStringValidator.class or the newer style, which is a list of specific type of Maps (validated by RasConstraintsTypeValidator.class). The value must be in one or the other format.Old style Config.TOPOLOGY_RAS_CONSTRAINTS (ListOfListOfString) specified a list of components that cannot co-exist on the same Worker.
New style Config.TOPOLOGY_RAS_CONSTRAINTS is map where each component has a list of other incompatible components (which serves the same function as the old style configuration) and optional number that specifies the maximum co-location count for the component on a node.
comp-1 cannot exist on same worker as comp-2 or comp-3, and at most "2" comp-1 on same node
comp-2 and comp-4 cannot be on same worker (missing comp-1 is implied from comp-1 constraint)
{ "comp-1": { "maxNodeCoLocationCnt": 2, "incompatibleComponents": ["comp-2", "comp-3" ] }, "comp-2": { "incompatibleComponents": [ "comp-4" ] } }
- See Also:
-
TOPOLOGY_SPREAD_COMPONENTS
Deprecated.Array of components that scheduler should try to place on separate hosts when using the constraint solver strategy or the multi-tenant scheduler. Note that this configuration can be specified in TOPOLOGY_RAS_CONSTRAINTS using the "maxNodeCoLocationCnt" map entry with value of 1.- See Also:
-
TOPOLOGY_RAS_CONSTRAINT_MAX_STATE_SEARCH
The maximum number of states that will be searched looking for a solution in resource aware strategies, e.g. in BaseResourceAwareStrategy.- See Also:
-
TOPOLOGY_RAS_ONE_EXECUTOR_PER_WORKER
- See Also:
-
TOPOLOGY_RAS_ONE_COMPONENT_PER_WORKER
Whether to limit each worker to one component. This is useful for debugging topologies to clearly identify workers that are slow/crashing and for estimating resource requirements and capacity. If both TOPOLOGY_RAS_ONE_EXECUTOR_PER_WORKER and TOPOLOGY_RAS_ONE_COMPONENT_PER_WORKER are enabled, TOPOLOGY_RAS_ONE_COMPONENT_PER_WORKER is ignored.- See Also:
-
TOPOLOGY_RAS_CONSTRAINT_MAX_TIME_SECS
The maximum number of seconds to spend scheduling a topology using resource aware strategies, e.g. in BaseResourceAwareStrategy. Null means no limit.- See Also:
-
TOPOLOGY_SCHEDULER_FAVORED_NODES
A list of host names that this topology would prefer to be scheduled on (no guarantee is given though). This is intended for debugging only.Favored nodes are moved to the front of the node selection list. If the same node is also present in
TOPOLOGY_SCHEDULER_UNFAVORED_NODES
then the node is considered only as a favored node and is removed from the unfavored list.- See Also:
-
TOPOLOGY_SCHEDULER_UNFAVORED_NODES
A list of host names that this topology would prefer to NOT be scheduled on (no guarantee is given though). This is intended for debugging only.Unfavored nodes are moved to the end of the node selection list. If the same node is also present in
TOPOLOGY_SCHEDULER_FAVORED_NODES
then the node is considered only as a favored node and is removed from the unfavored list.- See Also:
-
TOPOLOGY_ACKER_EXECUTORS
How many executors to spawn for ackers.1. If not setting this variable or setting it as null, a. If RAS is not used: Nimbus will set it to
TOPOLOGY_WORKERS
. b. If RAS is used: Nimbus will set it to (the estimate number of workers *TOPOLOGY_RAS_ACKER_EXECUTORS_PER_WORKER
).TOPOLOGY_RAS_ACKER_EXECUTORS_PER_WORKER
is default to be 1 if not set. 2. If this variable is set to 0, then Storm will immediately ack tuples as soon as they come off the spout, effectively disabling reliability. 3. If this variable is set to a positive integer, Storm will not honorTOPOLOGY_RAS_ACKER_EXECUTORS_PER_WORKER
setting. Instead, nimbus will set it as (this variable / estimate num of workers).- See Also:
-
TOPOLOGY_RAS_ACKER_EXECUTORS_PER_WORKER
How many ackers to put in when launching a new worker until we run out of ackers.This setting is RAS specific. If
TOPOLOGY_ACKER_EXECUTORS
is not configured, this setting will be used to calculateTOPOLOGY_ACKER_EXECUTORS
. IfTOPOLOGY_ACKER_EXECUTORS
is configured, nimbus will ignore this and set it as (TOPOLOGY_ACKER_EXECUTORS
/ estimate num of workers).- See Also:
-
TOPOLOGY_EVENT_LOGGER_REGISTER
A list of classes implementing IEventLogger (See storm.yaml.example for exact config format). Each listed class will be routed all the events sampled from emitting tuples. If there's no class provided to the option, default event logger will be initialized and used unless you disable event logger executor.Note that EventLoggerBolt takes care of all the implementations of IEventLogger, hence registering many implementations (especially they're implemented as 'blocking' manner) would slow down overall topology.
- See Also:
-
TOPOLOGY_EVENTLOGGER_EXECUTORS
How many executors to spawn for event logger.By setting it as null, Storm will set the number of eventlogger executors to be equal to the number of workers configured for this topology (or the estimated number of workers if the Resource Aware Scheduler is used). If this variable is set to 0, event logging will be disabled.
- See Also:
-
TOPOLOGY_MESSAGE_TIMEOUT_SECS
The maximum amount of time given to the topology to fully process a message emitted by a spout. If the message is not acked within this time frame, Storm will fail the message on the spout. Some spouts implementations will then replay the message at a later time.- See Also:
-
TOPOLOGY_KRYO_REGISTER
A list of serialization registrations for Kryo ( https://github.com/EsotericSoftware/kryo ), the underlying serialization framework for Storm. A serialization can either be the name of a class (in which case Kryo will automatically create a serializer for the class that saves all the object's fields), or an implementation of com.esotericsoftware.kryo.Serializer.See Kryo's documentation for more information about writing custom serializers.
- See Also:
-
TOPOLOGY_KRYO_DECORATORS
A list of classes that customize storm's kryo instance during start-up. Each listed class name must implement IKryoDecorator. During start-up the listed class is instantiated with 0 arguments, then its 'decorate' method is called with storm's kryo instance as the only argument.- See Also:
-
TOPOLOGY_KRYO_FACTORY
Class that specifies how to create a Kryo instance for serialization. Storm will then apply topology.kryo.register and topology.kryo.decorators on top of this. The default implementation implements topology.fall.back.on.java.serialization and turns references off.- See Also:
-
TOPOLOGY_SKIP_MISSING_KRYO_REGISTRATIONS
Whether or not Storm should skip the loading of kryo registrations for which it does not know the class or have the serializer implementation. Otherwise, the task will fail to load and will throw an error at runtime. The use case of this is if you want to declare your serializations on the storm.yaml files on the cluster rather than every single time you submit a topology. Different applications may use different serializations and so a single application may not have the code for the other serializers used by other apps. By setting this config to true, Storm will ignore that it doesn't have those other serializations rather than throw an error.- See Also:
-
TOPOLOGY_STATE_KRYO_REGISTER
List of classes to register during state serialization.- See Also:
-
TOPOLOGY_METRICS_CONSUMER_REGISTER
A list of classes implementing IMetricsConsumer (See storm.yaml.example for exact config format). Each listed class will be routed all the metrics data generated by the storm metrics API. Each listed class maps 1:1 to a system bolt named __metrics_ClassName#N, and it's parallelism is configurable.- See Also:
-
TOPOLOGY_SERIALIZED_MESSAGE_SIZE_METRICS
Enable tracking of network message byte counts per source-destination task. This is off by default as it creates tasks^2 metric values, but is useful for debugging as it exposes data skew when tuple sizes are uneven.- See Also:
-
TOPOLOGY_WORKER_METRICS
A map of metric name to class name implementing IMetric that will be created once per worker JVM.- See Also:
-
WORKER_METRICS
A map of metric name to class name implementing IMetric that will be created once per worker JVM.- See Also:
-
TOPOLOGY_MAX_TASK_PARALLELISM
The maximum parallelism allowed for a component in this topology. This configuration is typically used in testing to limit the number of threads spawned in local mode.- See Also:
-
TOPOLOGY_MAX_SPOUT_PENDING
The maximum number of tuples that can be pending on a spout task at any given time. This config applies to individual tasks, not to spouts or topologies as a whole.A pending tuple is one that has been emitted from a spout but has not been acked or failed yet. Note that this config parameter has no effect for unreliable spouts that don't tag their tuples with a message id.
- See Also:
-
TOPOLOGY_SLEEP_SPOUT_WAIT_STRATEGY_TIME_MS
The amount of milliseconds the SleepEmptyEmitStrategy should sleep for.- See Also:
-
TOPOLOGY_STATE_SYNCHRONIZATION_TIMEOUT_SECS
The maximum amount of time a component gives a source of state to synchronize before it requests synchronization again.- See Also:
-
TOPOLOGY_STATS_SAMPLE_RATE
The percentage of tuples to sample to produce stats for a task.- See Also:
-
TOPOLOGY_BUILTIN_METRICS_BUCKET_SIZE_SECS
The time period that builtin metrics data in bucketed into.- See Also:
-
TOPOLOGY_FALL_BACK_ON_JAVA_SERIALIZATION
Whether or not to use Java serialization in a topology. Default is set false for security reasons.- See Also:
-
TOPOLOGY_WORKER_CHILDOPTS
Topology-specific options for the worker child process. This is used in addition to WORKER_CHILDOPTS.- See Also:
-
TOPOLOGY_WORKER_GC_CHILDOPTS
Topology-specific options GC for the worker child process. This overrides WORKER_GC_CHILDOPTS.- See Also:
-
TOPOLOGY_WORKER_LOGWRITER_CHILDOPTS
Topology-specific options for the logwriter process of a worker.- See Also:
-
TOPOLOGY_CLASSPATH
Topology-specific classpath for the worker child process. This is combined to the usual classpath.- See Also:
-
TOPOLOGY_CLASSPATH_BEGINNING
Topology-specific classpath for the worker child process. This will be *prepended* to the usual classpath, meaning it can override the Storm classpath. This is for debugging purposes, and is disabled by default. To allow topologies to be submitted with user-first classpaths, set the storm.topology.classpath.beginning.enabled config to true.- See Also:
-
TOPOLOGY_ENVIRONMENT
Topology-specific environment variables for the worker child process. This is added to the existing environment (that of the supervisor)- See Also:
-
TOPOLOGY_BOLTS_WINDOW_LENGTH_COUNT
- See Also:
-
TOPOLOGY_BOLTS_WINDOW_LENGTH_DURATION_MS
- See Also:
-
TOPOLOGY_BOLTS_SLIDING_INTERVAL_COUNT
- See Also:
-
TOPOLOGY_BOLTS_SLIDING_INTERVAL_DURATION_MS
- See Also:
-
TOPOLOGY_BOLTS_LATE_TUPLE_STREAM
Bolt-specific configuration for windowed bolts to specify the name of the stream on which late tuples are going to be emitted. This configuration should only be used from the BaseWindowedBolt.withLateTupleStream builder method, and not as global parameter, otherwise IllegalArgumentException is going to be thrown.- See Also:
-
TOPOLOGY_BOLTS_TUPLE_TIMESTAMP_MAX_LAG_MS
Bolt-specific configuration for windowed bolts to specify the maximum time lag of the tuple timestamp in milliseconds. It means that the tuple timestamps cannot be out of order by more than this amount. This config will be effective only ifTimestampExtractor
is specified.- See Also:
-
TOPOLOGY_BOLTS_WATERMARK_EVENT_INTERVAL_MS
- See Also:
-
TOPOLOGY_BOLTS_MESSAGE_ID_FIELD_NAME
- See Also:
-
TOPOLOGY_TRANSACTIONAL_ID
This config is available for TransactionalSpouts, and contains the id ( a String) for the transactional topology. This id is used to store the state of the transactional topology in Zookeeper.- See Also:
-
TOPOLOGY_AUTO_TASK_HOOKS
A list of task hooks that are automatically added to every spout and bolt in the topology. An example of when you'd do this is to add a hook that integrates with your internal monitoring system. These hooks are instantiated using the zero-arg constructor.- See Also:
-
TOPOLOGY_EXECUTOR_RECEIVE_BUFFER_SIZE
The size of the receive queue for each executor.- See Also:
-
TOPOLOGY_TRANSFER_BUFFER_SIZE
The size of the transfer queue for each worker.- See Also:
-
TOPOLOGY_TRANSFER_BATCH_SIZE
The size of the transfer queue for each worker.- See Also:
-
TOPOLOGY_TICK_TUPLE_FREQ_SECS
How often a tick tuple from the "__system" component and "__tick" stream should be sent to tasks. Meant to be used as a component-specific configuration.- See Also:
-
TOPOLOGY_PRODUCER_BATCH_SIZE
The number of tuples to batch before sending to the destination executor.- See Also:
-
TOPOLOGY_EXECUTOR_OVERFLOW_LIMIT
If number of items in task's overflowQ exceeds this, new messages coming from other workers to this task will be dropped This prevents OutOfMemoryException that can occur in rare scenarios in the presence of BackPressure. This affects only inter-worker messages. Messages originating from within the same worker will not be dropped.- See Also:
-
TOPOLOGY_BACKPRESSURE_CHECK_MILLIS
How often a worker should check and notify upstream workers about its tasks that are no longer experiencing BP and able to receive new messages.- See Also:
-
TOPOLOGY_BATCH_FLUSH_INTERVAL_MILLIS
How often to send flush tuple to the executors for flushing out batched events.- See Also:
-
TOPOLOGY_WORKER_SHARED_THREAD_POOL_SIZE
The size of the shared thread pool for worker tasks to make use of. The thread pool can be accessed via the TopologyContext.- See Also:
-
TOPOLOGY_ERROR_THROTTLE_INTERVAL_SECS
The interval in seconds to use for determining whether to throttle error reported to Zookeeper. For example, an interval of 10 seconds with topology.max.error.report.per.interval set to 5 will only allow 5 errors to be reported to Zookeeper per task for every 10 second interval of time.- See Also:
-
TOPOLOGY_MAX_ERROR_REPORT_PER_INTERVAL
See doc forTOPOLOGY_ERROR_THROTTLE_INTERVAL_SECS
.- See Also:
-
TOPOLOGY_TRIDENT_BATCH_EMIT_INTERVAL_MILLIS
How often a batch can be emitted in a Trident topology.- See Also:
-
TOPOLOGY_TRIDENT_WINDOWING_INMEMORY_CACHE_LIMIT
Maximum number of tuples that can be stored inmemory cache in windowing operators for fast access without fetching them from store.- See Also:
-
STORM_ID
The id assigned to a running topology. The id is the storm name with a unique nonce appended.- See Also:
-
TOPOLOGY_NAME
Name of the topology. This config is automatically set by Storm when the topology is submitted.- See Also:
-
TOPOLOGY_SUBMITTER_PRINCIPAL
The principal who submitted a topology.- See Also:
-
TOPOLOGY_SUBMITTER_USER
The local user name of the user who submitted a topology.- See Also:
-
TOPOLOGY_AUTO_CREDENTIALS
A list of IAutoCredentials that the topology should load and use.- See Also:
-
TOPOLOGY_SHELLBOLT_MAX_PENDING
Max pending tuples in one ShellBolt.- See Also:
-
TOPOLOGY_SUBPROCESS_TIMEOUT_SECS
How long a subprocess can go without heartbeating before the ShellSpout/ShellBolt tries to suicide itself.- See Also:
-
TOPOLOGY_LOGGING_SENSITIVITY
Topology central logging sensitivity to determine who has access to logs in central logging system. The possible values are: S0 - Public (open to all users on grid) S1 - Restricted S2 - Confidential S3 - Secret (default.)- See Also:
-
TOPOLOGY_LOGGING_CONFIG_FILE
Log file the user can use to configure Log4j2. Can be a resource in the jar (specified with classpath:/path/to/resource) or a file. This configuration is applied in addition to the regular worker log4j2 configuration. The configs are merged according to the rules here: https://logging.apache.org/log4j/2.x/manual/configuration.html#CompositeConfiguration- See Also:
-
TOPOLOGY_PRIORITY
Sets the priority for a topology.- See Also:
-
TRANSACTIONAL_ZOOKEEPER_ROOT
The root directory in ZooKeeper for metadata about TransactionalSpouts.- See Also:
-
TRANSACTIONAL_ZOOKEEPER_SERVERS
The list of zookeeper servers in which to keep the transactional state. If null (which is default), will use storm.zookeeper.servers- See Also:
-
TRANSACTIONAL_ZOOKEEPER_PORT
The port to use to connect to the transactional zookeeper servers. If null (which is default), will use storm.zookeeper.port- See Also:
-
STORM_DO_AS_USER
The user as which the nimbus client should be acquired to perform the operation.- See Also:
-
TOPOLOGY_ISOLATED_MACHINES
The maximum number of machines that should be used by this topology. This configuration can be used to isolate topologies from each other. Seeorg.apache.storm.scheduler.multitenant.MultitenantScheduler
. Round Robin Strategy uses this value to avoid spreading a topology too thinly over a large number of machines - avoiding the extreme case where the topology would be spread over all workers and thus deny scheduling of other topologies. Round Robin scheduling will occupy all the workers on this limited number of machines, forcing other topologies to be scheduled on other machines; thus isolating the topology from other topologies. Setstorm.scheduler
toorg.apache.storm.scheduler.multitenant.MultitenantScheduler
Alternatively setstorm.scheduler
toorg.apache.storm.scheduler.resource.ResourceAwareScheduler
usingTOPOLOGY_SCHEDULER_STRATEGY
set toorg.apache.storm.scheduler.resource.strategies.scheduling.RoundRobinResourceAwareStrategy
- See Also:
-
TOPOLOGY_SPOUT_WAIT_STRATEGY
A class that implements a wait strategy for spout. Waiting is triggered in one of two conditions:1. nextTuple emits no tuples 2. The spout has hit maxSpoutPending and can't emit any more tuples
This class must implement
IWaitStrategy
.- See Also:
-
TOPOLOGY_SPOUT_WAIT_PARK_MICROSEC
Configures park time for WaitStrategyPark for spout. If set to 0, returns immediately (i.e busy wait).- See Also:
-
TOPOLOGY_SPOUT_WAIT_PROGRESSIVE_LEVEL1_COUNT
Configures number of iterations to spend in level 1 of WaitStrategyProgressive, before progressing to level 2.- See Also:
-
TOPOLOGY_SPOUT_WAIT_PROGRESSIVE_LEVEL2_COUNT
Configures number of iterations to spend in level 2 of WaitStrategyProgressive, before progressing to level 3.- See Also:
-
TOPOLOGY_SPOUT_WAIT_PROGRESSIVE_LEVEL3_SLEEP_MILLIS
Configures sleep time for WaitStrategyProgressive.- See Also:
-
TOPOLOGY_BOLT_WAIT_STRATEGY
Selects the Bolt's Wait Strategy to use when there are no incoming msgs. Used to trade off latency vs CPU usage. This class must implementIWaitStrategy
.- See Also:
-
TOPOLOGY_BOLT_WAIT_PARK_MICROSEC
Configures park time for WaitStrategyPark. If set to 0, returns immediately (i.e busy wait).- See Also:
-
TOPOLOGY_BOLT_WAIT_PROGRESSIVE_LEVEL1_COUNT
Configures number of iterations to spend in level 1 of WaitStrategyProgressive, before progressing to level 2.- See Also:
-
TOPOLOGY_BOLT_WAIT_PROGRESSIVE_LEVEL2_COUNT
Configures number of iterations to spend in level 2 of WaitStrategyProgressive, before progressing to level 3.- See Also:
-
TOPOLOGY_BOLT_WAIT_PROGRESSIVE_LEVEL3_SLEEP_MILLIS
Configures sleep time for WaitStrategyProgressive.- See Also:
-
TOPOLOGY_BACKPRESSURE_WAIT_STRATEGY
A class that implements a wait strategy for an upstream component (spout/bolt) trying to write to a downstream component whose recv queue is full1. nextTuple emits no tuples 2. The spout has hit maxSpoutPending and can't emit any more tuples
This class must implement
IWaitStrategy
.- See Also:
-
TOPOLOGY_BACKPRESSURE_WAIT_PARK_MICROSEC
Configures park time if using WaitStrategyPark for BackPressure. If set to 0, returns immediately (i.e busy wait).- See Also:
-
TOPOLOGY_BACKPRESSURE_WAIT_PROGRESSIVE_LEVEL3_SLEEP_MILLIS
Configures sleep time if using WaitStrategyProgressive for BackPressure.- See Also:
-
TOPOLOGY_BACKPRESSURE_WAIT_PROGRESSIVE_LEVEL1_COUNT
Configures steps used to determine progression to the next level of wait .. if using WaitStrategyProgressive for BackPressure.- See Also:
-
TOPOLOGY_BACKPRESSURE_WAIT_PROGRESSIVE_LEVEL2_COUNT
Configures steps used to determine progression to the next level of wait .. if using WaitStrategyProgressive for BackPressure.- See Also:
-
TOPOLOGY_SPOUT_RECVQ_SKIPS
Check recvQ after every N invocations of Spout's nextTuple() [when ACKing is disabled]. Spouts receive very few msgs if ACK is disabled. This avoids checking the recvQ after each nextTuple().- See Also:
-
TOPOLOGY_MIN_REPLICATION_COUNT
Minimum number of nimbus hosts where the code must be replicated before leader nimbus is allowed to perform topology activation tasks like setting up heartbeats/assignments and marking the topology as active. default is 0.- See Also:
-
TOPOLOGY_MAX_REPLICATION_WAIT_TIME_SEC
Maximum wait time for the nimbus host replication to achieve the nimbus.min.replication.count. Once this time is elapsed nimbus will go ahead and perform topology activation tasks even if required nimbus.min.replication.count is not achieved. The default is 0 seconds, a value of -1 indicates to wait for ever.- See Also:
-
PACEMAKER_SERVERS
The list of servers that Pacemaker is running on.- See Also:
-
PACEMAKER_PORT
The port Pacemaker should run on. Clients should connect to this port to submit or read heartbeats.- See Also:
-
PACEMAKER_CLIENT_MAX_THREADS
The maximum number of threads that should be used by the Pacemaker client. When Pacemaker gets loaded it will spawn new threads, up to this many total, to handle the load.- See Also:
-
PACEMAKER_AUTH_METHOD
This should be one of "DIGEST", "KERBEROS", or "NONE" Determines the mode of authentication the pacemaker server and client use. The client must either match the server, or be NONE. In the case of NONE, no authentication is performed for the client, and if the server is running with DIGEST or KERBEROS, the client can only write to the server (no reads). This is intended to provide a primitive form of access-control.- See Also:
-
PACEMAKER_THRIFT_MESSAGE_SIZE_MAX
Pacemaker Thrift Max Message Size (bytes).- See Also:
-
STORM_GROUP_MAPPING_SERVICE_CACHE_DURATION_SECS
Max no.of seconds group mapping service will cache user groups- See Also:
-
DRPC_SERVERS
List of DRPC servers so that the DRPCSpout knows who to talk to.- See Also:
-
DRPC_INVOCATIONS_PORT
This port on Storm DRPC is used by DRPC topologies to receive function invocations and send results back.- See Also:
-
STORM_NIMBUS_RETRY_TIMES
The number of times to retry a Nimbus operation.- See Also:
-
STORM_NIMBUS_RETRY_INTERVAL
The starting interval between exponential backoff retries of a Nimbus operation.- See Also:
-
STORM_NIMBUS_RETRY_INTERVAL_CEILING
The ceiling of the interval between retries of a client connect to Nimbus operation.- See Also:
-
NIMBUS_THRIFT_TRANSPORT_PLUGIN
The Nimbus transport plug-in for Thrift client/server communication.- See Also:
-
NIMBUS_THRIFT_PORT
Which port the Thrift interface of Nimbus should run on. Clients should connect to this port to upload jars and submit topologies.- See Also:
-
NIMBUS_QUEUE_SIZE
Nimbus thrift server queue size, default is 100000. This is the request queue size , when there are more requests than number of threads to serve the requests, those requests will be queued to this queue. If the request queue size > this config, then the incoming requests will be rejected.- See Also:
-
NIMBUS_LOCAL_ASSIGNMENTS_BACKEND_CLASS
Nimbus assignments backend for storing local assignments. We will use it to store physical plan and runtime storm ids.- See Also:
-
NIMBUS_THRIFT_THREADS
The number of threads that should be used by the nimbus thrift server.- See Also:
-
NIMBUS_THRIFT_MAX_BUFFER_SIZE
The maximum buffer size thrift should use when reading messages.- See Also:
-
STORM_THRIFT_SOCKET_TIMEOUT_MS
How long before a Thrift Client socket hangs before timeout and restart the socket.- See Also:
-
DRPC_THRIFT_TRANSPORT_PLUGIN
The DRPC transport plug-in for Thrift client/server communication.- See Also:
-
DRPC_PORT
This port is used by Storm DRPC for receiving DPRC requests from clients.- See Also:
-
DRPC_QUEUE_SIZE
DRPC thrift server queue size.- See Also:
-
DRPC_WORKER_THREADS
DRPC thrift server worker threads.- See Also:
-
DRPC_MAX_BUFFER_SIZE
The maximum buffer size thrift should use when reading messages for DRPC.- See Also:
-
DRPC_INVOCATIONS_THRIFT_TRANSPORT_PLUGIN
The DRPC invocations transport plug-in for Thrift client/server communication.- See Also:
-
DRPC_INVOCATIONS_THREADS
DRPC invocations thrift server worker threads.- See Also:
-
STORM_GROUP_MAPPING_SERVICE_PARAMS
Initialization parameters for the group mapping service plugin. Provides a way for aSTORM_GROUP_MAPPING_SERVICE_PROVIDER_PLUGIN
implementation to access optional settings.- See Also:
-
STORM_THRIFT_TRANSPORT_PLUGIN
The default transport plug-in for Thrift client/server communication.- See Also:
-
SUPERVISOR_WORKER_TIMEOUT_SECS
How long a worker can go without heartbeating before the supervisor tries to restart the worker process. Can be overridden byTOPOLOGY_WORKER_TIMEOUT_SECS
, if set.- See Also:
-
WORKER_MAX_TIMEOUT_SECS
Enforce maximum onTOPOLOGY_WORKER_TIMEOUT_SECS
.- See Also:
-
TOPOLOGY_WORKER_TIMEOUT_SECS
Topology configurable worker heartbeat timeout before the supervisor tries to restart the worker process. Maximum value constrained byWORKER_MAX_TIMEOUT_SECS
. When topology timeout is greater, the following configs are effectively overridden:SUPERVISOR_WORKER_TIMEOUT_SECS
, SUPERVISOR_WORKER_START_TIMEOUT_SECS, NIMBUS_TASK_TIMEOUT_SECS and NIMBUS_TASK_LAUNCH_SECS.- See Also:
-
SUPERVISOR_WORKER_SHUTDOWN_SLEEP_SECS
How many seconds to allow for graceful worker shutdown when killing workers before resorting to force kill. If a worker fails to shut down gracefully within this delay, it will either suicide or be forcibly killed by the supervisor.- See Also:
-
STORM_ZOOKEEPER_SERVERS
A list of hosts of ZooKeeper servers used to manage the cluster.- See Also:
-
STORM_ZOOKEEPER_PORT
The port Storm will use to connect to each of the ZooKeeper servers.- See Also:
-
STORM_ZOOKEEPER_SUPERACL
This is part of a temporary workaround to a ZK bug, it is the 'scheme:acl' for the user Nimbus and Supervisors use to authenticate with ZK.- See Also:
-
STORM_ZOOKEEPER_DRPC_ACL
The ACL of the drpc user in zookeeper so the drpc servers can verify worker tokens.Should be in the form 'scheme:acl' just like STORM_ZOOKEEPER_SUPERACL.
- See Also:
-
STORM_ZOOKEEPER_TOPOLOGY_AUTH_SCHEME
The topology Zookeeper authentication scheme to use, e.g. "digest". It is the internal config and user shouldn't set it.- See Also:
-
STORM_META_SERIALIZATION_DELEGATE
The delegate for serializing metadata, should be used for serialized objects stored in zookeeper and on disk. This is NOT used for compressing serialized tuples sent between topologies.- See Also:
-
TOPOLOGY_METRICS_REPORTERS
Configure the topology metrics reporters to be used on workers.- See Also:
-
STORM_TOPOLOGY_METRICS_SYSTEM_REPORTERS
A list of system metrics reporters that will get added to each topology.- See Also:
-
STORM_METRICS_REPORTERS
Deprecated, for removal: This API element is subject to removal in a future version.Configure the topology metrics reporters to be used on workers. UseTOPOLOGY_METRICS_REPORTERS
instead.- See Also:
-
CLIENT_BLOBSTORE
What blobstore implementation the storm client should use.- See Also:
-
BLOBSTORE_DIR
What directory to use for the blobstore. The directory is expected to be an absolute path when using HDFS blobstore, for LocalFsBlobStore it could be either absolute or relative. If the setting is a relative directory, it is relative to root directory of Storm installation.- See Also:
-
BLOBSTORE_CLEANUP_ENABLE
Enable the blobstore cleaner. Certain blobstores may only want to run the cleaner on one daemon. Currently Nimbus handles setting this.- See Also:
-
BLOBSTORE_HDFS_PRINCIPAL
Deprecated, for removal: This API element is subject to removal in a future version.principal for nimbus/supervisor to use to access secure hdfs for the blobstore. The format is generally "primary/instance@REALM", where "instance" field is optional. If the instance field of the principal is the string "_HOST", it will be replaced with the host name of the server the daemon is running on (by callinggetBlobstoreHDFSPrincipal(Map conf)
method). Note: UseSTORM_HDFS_LOGIN_PRINCIPAL
instead.- See Also:
-
BLOBSTORE_HDFS_KEYTAB
Deprecated.keytab for nimbus/supervisor to use to access secure hdfs for the blobstore.- See Also:
-
STORM_BLOBSTORE_REPLICATION_FACTOR
Set replication factor for a blob in HDFS Blobstore Implementation.- See Also:
-
STORM_HDFS_LOGIN_PRINCIPAL
The principal for nimbus/supervisor to use to access secure hdfs. The format is generally "primary/instance@REALM", where "instance" field is optional. If the instance field of the principal is the string "_HOST", it will be replaced with the host name of the server the daemon is running on (by callinggetHdfsPrincipal(java.util.Map<java.lang.String, java.lang.Object>)
method).- See Also:
-
STORM_HDFS_LOGIN_KEYTAB
The keytab for nimbus/supervisor to use to access secure hdfs.- See Also:
-
STORM_LOCAL_HOSTNAME
The hostname the supervisors/workers should report to nimbus. If unset, Storm will get the hostname to report by callingInetAddress.getLocalHost().getCanonicalHostName()
.You should set this config when you don't have a DNS which supervisors/workers can utilize to find each other based on hostname got from calls to
InetAddress.getLocalHost().getCanonicalHostName()
.- See Also:
-
NIMBUS_SEEDS
List of seed nimbus hosts to use for leader nimbus discovery.- See Also:
-
NIMBUS_USERS
A list of users that are the only ones allowed to run user operation on storm cluster. To use this set nimbus.authorizer to org.apache.storm.security.auth.authorizer.SimpleACLAuthorizer- See Also:
-
NIMBUS_GROUPS
A list of groups , users belong to these groups are the only ones allowed to run user operation on storm cluster. To use this set nimbus.authorizer to org.apache.storm.security.auth.authorizer.SimpleACLAuthorizer- See Also:
-
STORM_CLUSTER_MODE
The mode this Storm cluster is running in. Either "distributed" or "local".- See Also:
-
STORM_ZOOKEEPER_ROOT
The root location at which Storm stores data in ZooKeeper.- See Also:
-
STORM_ZOOKEEPER_TOPOLOGY_AUTH_PAYLOAD
A string representing the payload for topology Zookeeper authentication. It gets serialized using UTF-8 encoding during authentication.- See Also:
-
STORM_ZOOKEEPER_AUTH_SCHEME
The cluster Zookeeper authentication scheme to use, e.g. "digest". Defaults to no authentication.- See Also:
-
STORM_ZOOKEEPER_AUTH_PAYLOAD
A string representing the payload for cluster Zookeeper authentication. It gets serialized using UTF-8 encoding during authentication. Note that if this is set to something with a secret (as when using digest authentication) then it should only be set in the storm-cluster-auth.yaml file. This file storm-cluster-auth.yaml should then be protected with appropriate permissions that deny access from workers.- See Also:
-
STORM_NETWORK_TOPOGRAPHY_PLUGIN
What Network Topography detection classes should we use. Given a list of supervisor hostnames (or IP addresses), this class would return a list of rack names that correspond to the supervisors. This information is stored in Cluster.java, and is used in the resource aware scheduler.- See Also:
-
WORKER_GC_CHILDOPTS
The jvm opts provided to workers launched by this supervisor for GC. All "%ID%" substrings are replaced with an identifier for this worker. Because the JVM complains about multiple GC opts the topology can override this default value by setting topology.worker.gc.childopts.- See Also:
-
WORKER_CHILDOPTS
The jvm opts provided to workers launched by this supervisor. All "%ID%", "%WORKER-ID%", "%TOPOLOGY-ID%", "%WORKER-PORT%" and "%HEAP-MEM%" substrings are replaced with: %ID% -> port (for backward compatibility), %WORKER-ID% -> worker-id, %TOPOLOGY-ID% -> topology-id, %WORKER-PORT% -> port. %HEAP-MEM% -> mem-onheap.- See Also:
-
WORKER_HEAP_MEMORY_MB
The default heap memory size in MB per worker, used in the jvm -Xmx opts for launching the worker.- See Also:
-
SUPERVISOR_MEMORY_CAPACITY_MB
The total amount of memory (in MiB) a supervisor is allowed to give to its workers. A default value will be set for this config if user does not override- See Also:
-
SUPERVISOR_CPU_CAPACITY
The total amount of CPU resources a supervisor is allowed to give to its workers. By convention 1 cpu core should be about 100, but this can be adjusted if needed using 100 makes it simple to set the desired value to the capacity measurement for single threaded bolts. A default value will be set for this config if user does not override- See Also:
-
SUPERVISOR_THRIFT_PORT
- See Also:
-
SUPERVISOR_THRIFT_TRANSPORT_PLUGIN
- See Also:
-
SUPERVISOR_QUEUE_SIZE
- See Also:
-
SUPERVISOR_THRIFT_THREADS
- See Also:
-
SUPERVISOR_THRIFT_MAX_BUFFER_SIZE
- See Also:
-
SUPERVISOR_THRIFT_SOCKET_TIMEOUT_MS
How long before a supervisor Thrift Client socket hangs before timeout and restart the socket.- See Also:
-
SUPERVISOR_RESOURCES_MAP
A map of resources the Supervisor has e.g {"cpu.pcore.percent" : 200.0. "onheap.memory.mb": 256.0, "gpu.count" : 2.0 }- See Also:
-
STORM_LOCAL_MODE_ZMQ
Whether or not to use ZeroMQ for messaging in local mode. If this is set to false, then Storm will use a pure-Java messaging system. The purpose of this flag is to make it easy to run Storm in local mode by eliminating the need for native dependencies, which can be difficult to install.Defaults to false.
- See Also:
-
STORM_MESSAGING_TRANSPORT
The transporter for communication among Storm tasks.- See Also:
-
STORM_MESSAGING_NETTY_AUTHENTICATION
Netty based messaging: Is authentication required for Netty messaging from client worker process to server worker process. See https://issues.apache.org/jira/browse/STORM-348 for more details- See Also:
-
STORM_MESSAGING_NETTY_BUFFER_SIZE
Netty based messaging: The buffer size for send/recv buffer.- See Also:
-
STORM_MESSAGING_NETTY_BUFFER_HIGH_WATERMARK
Netty based messaging: The netty write buffer high watermark in bytes.If the number of bytes queued in the netty's write buffer exceeds this value, the netty
Channel.isWritable()
will start to returnfalse
. The client will wait until the value falls below the low water mark.- See Also:
-
STORM_MESSAGING_NETTY_BUFFER_LOW_WATERMARK
Netty based messaging: The netty write buffer low watermark in bytes.Once the number of bytes queued in the write buffer exceeded the high water mark and then dropped down below this value, the netty
Channel.isWritable()
will start to return true.- See Also:
-
STORM_MESSAGING_NETTY_SOCKET_BACKLOG
Netty based messaging: Sets the backlog value to specify when the channel binds to a local address.- See Also:
-
STORM_MESSAGING_NETTY_SERVER_WORKER_THREADS
Netty based messaging: The # of worker threads for the server.- See Also:
-
STORM_NETTY_MESSAGE_BATCH_SIZE
If the Netty messaging layer is busy, the Netty client will try to batch message as more as possible up to the size of STORM_NETTY_MESSAGE_BATCH_SIZE bytes.- See Also:
-
STORM_MESSAGING_NETTY_MIN_SLEEP_MS
Netty based messaging: The min # of milliseconds that a peer will wait.- See Also:
-
STORM_MESSAGING_NETTY_MAX_SLEEP_MS
Netty based messaging: The max # of milliseconds that a peer will wait.- See Also:
-
STORM_MESSAGING_NETTY_CLIENT_WORKER_THREADS
Netty based messaging: The # of worker threads for the client.- See Also:
-
SUPERVISOR_RUN_WORKER_AS_USER
Should the supervior try to run the worker as the lauching user or not. Defaults to false.- See Also:
-
SUPERVISOR_WORKER_HEARTBEATS_MAX_TIMEOUT_SECS
max timeout for supervisor reported heartbeats when master gains leadership.- See Also:
-
DISABLE_SYMLINKS
On some systems (windows for example) symlinks require special privileges that not everyone wants to grant a headless user. You can completely disable the use of symlinks by setting this config to true, but by doing so you may also lose some features from storm. For example the blobstore feature does not currently work without symlinks enabled.- See Also:
-
STORM_PRINCIPAL_TO_LOCAL_PLUGIN
The plugin that will convert a principal to a local user.- See Also:
-
STORM_GROUP_MAPPING_SERVICE_PROVIDER_PLUGIN
The plugin that will provide user groups service.- See Also:
-
NIMBUS_CREDENTIAL_RENEWERS
A list of credential renewers that nimbus should load.- See Also:
-
NIMBUS_AUTO_CRED_PLUGINS
A list of plugins that nimbus should load during submit topology to populate credentials on user's behalf.- See Also:
-
NIMBUS_SUPERVISOR_USERS
A list of users that run the supervisors and should be authorized to interact with nimbus as a supervisor would. To use this set nimbus.authorizer to org.apache.storm.security.auth.authorizer.SimpleACLAuthorizer.- See Also:
-
NIMBUS_DAEMON_USERS
A list of users that nimbus runs as and should be authorized to interact with the supervisor as nimbus would. To use this set supervisor.authorizer to org.apache.storm.security.auth.authorizer.SupervisorSimpleACLAuthorizer.- See Also:
-
NIMBUS_ADMINS
A list of users that are cluster admins and can run any command. To use this set nimbus.authorizer to org.apache.storm.security.auth.authorizer.SimpleACLAuthorizer- See Also:
-
NIMBUS_ADMINS_GROUPS
A list of groups that are cluster admins and can run any command.- See Also:
-
STORM_BLOBSTORE_ACL_VALIDATION_ENABLED
For secure mode we would want to turn on this config By default this is turned off assuming the default is insecure.- See Also:
-
STORM_BLOBSTORE_INPUTSTREAM_BUFFER_SIZE_BYTES
What buffer size to use for the blobstore uploads.- See Also:
-
STORM_BLOBSTORE_DEPENDENCY_JAR_UPLOAD_CHUNK_SIZE_BYTES
What chunk size to use for storm client to upload dependency jars.- See Also:
-
STORM_TOPOLOGY_SUBMISSION_NOTIFIER_PLUGIN
FQCN of a class that implementsISubmitterHook
@see ISubmitterHook for details.- See Also:
-
NIMBUS_IMPERSONATION_ACL
Impersonation user ACL config entries.- See Also:
-
NIMBUS_SCHEDULER_STRATEGY_CLASS_WHITELIST
A whitelist of the RAS scheduler strategies allowed by nimbus. Should be a list of fully-qualified class names or null to allow all.- See Also:
-
SUPERVISOR_WORKER_LAUNCHER
Full path to the worker-laucher executable that will be used to lauch workers when SUPERVISOR_RUN_WORKER_AS_USER is set to true.- See Also:
-
SUPERVISOR_WORKER_VERSION_CLASSPATH_MAP
Map a version of storm to a worker classpath that can be used to run it. This allows the supervisor to select an available version of storm that is compatible with what a topology was launched with.Only the major and minor version numbers are used, although this may change in the future. The code will first try to find a version that is the same or higher than the requested version, but with the same major version number. If it cannot it will fall back to using one with a lower minor version, but in some cases this might fail as some features may be missing.
Because of how this selection process works please don't include two releases with the same major and minor versions as it is undefined which will be selected. Also it is good practice to just include one release for each major version you want to support unless the minor versions are truly not compatible with each other. This is to avoid maintenance and testing overhead.
This config needs to be set on all supervisors and on nimbus. In general this can be the output of calling storm classpath on the version you want and adding in an entry for the config directory for that release. You should modify the storm.yaml of each of these versions to match the features and settings you want on the main version.
- See Also:
-
SUPERVISOR_WORKER_VERSION_MAIN_MAP
Map a version of storm to a worker's main class. In most cases storm should have correct defaults and just setting SUPERVISOR_WORKER_VERSION_CLASSPATH_MAP is enough.- See Also:
-
SUPERVISOR_WORKER_VERSION_LOGWRITER_MAP
Map a version of storm to a worker's logwriter class. In most cases storm should have correct defaults and just setting SUPERVISOR_WORKER_VERSION_CLASSPATH_MAP is enough.- See Also:
-
SUPERVISOR_WORKER_DEFAULT_VERSION
The version of storm to assume a topology should run as if not version is given by the client when submitting the topology.- See Also:
-
STORM_LOCAL_DIR
A directory on the local filesystem used by Storm for any local filesystem usage it needs. The directory must exist and the Storm daemons must have permission to read/write from this location. It could be either absolute or relative. If the setting is a relative directory, it is relative to root directory of Storm installation.- See Also:
-
STORM_WORKERS_ARTIFACTS_DIR
The workers-artifacts directory (where we place all workers' logs), can be either absolute or relative. By default, ${storm.log.dir}/workers-artifacts is where worker logs go. If the setting is a relative directory, it is relative to storm.log.dir.- See Also:
-
STORM_ZOOKEEPER_CONNECTION_TIMEOUT
The connection timeout for clients to ZooKeeper.- See Also:
-
STORM_ZOOKEEPER_SESSION_TIMEOUT
The session timeout for clients to ZooKeeper.- See Also:
-
STORM_ZOOKEEPER_RETRY_INTERVAL
The interval between retries of a Zookeeper operation.- See Also:
-
STORM_ZOOKEEPER_RETRY_INTERVAL_CEILING
The ceiling of the interval between retries of a Zookeeper operation.- See Also:
-
STORM_ZOOKEEPER_RETRY_TIMES
The number of times to retry a Zookeeper operation.- See Also:
-
STORM_CLUSTER_STATE_STORE
The ClusterState factory that worker will use to create a ClusterState to store state in. Defaults to ZooKeeper.- See Also:
-
WORKER_HEARTBEAT_FREQUENCY_SECS
How often this worker should heartbeat to the supervisor.- See Also:
-
EXECUTOR_METRICS_FREQUENCY_SECS
How often executor metrics should report to master, used for RPC heartbeat mode.- See Also:
-
TASK_HEARTBEAT_FREQUENCY_SECS
How often a task should heartbeat its status to the Pacamker. For 2.0 RPC heartbeat reporting, seeEXECUTOR_METRICS_FREQUENCY_SECS
.- See Also:
-
TASK_REFRESH_POLL_SECS
How often a task should sync its connections with other tasks (if a task is reassigned, the other tasks sending messages to it need to refresh their connections). In general though, when a reassignment happens other tasks will be notified almost immediately. This configuration is here just in case that notification doesn't come through.- See Also:
-
DRPC_AUTHORIZER_ACL
The Access Control List for the DRPC Authorizer.- See Also:
-
DRPC_AUTHORIZER_ACL_FILENAME
File name of the DRPC Authorizer ACL.- See Also:
-
DRPC_AUTHORIZER_ACL_STRICT
Whether the DRPCSimpleAclAuthorizer should deny requests for operations involving functions that have no explicit ACL entry. When set to false (the default) DRPC functions that have no entry in the ACL will be permitted, which is appropriate for a development environment. When set to true, explicit ACL entries are required for every DRPC function, and any request for functions will be denied.- See Also:
-
STORM_CGROUP_HIERARCHY_DIR
root directory of the storm cgroup hierarchy.- See Also:
-
NUM_STAT_BUCKETS
The number of Buckets.- See Also:
-
STORM_OCI_CGROUP_ROOT
The root of cgroup for oci to use. On RHEL7, it should be "/sys/fs/cgroup". -
TOPOLOGY_OCI_IMAGE
Specify the oci image to use. -
WORKER_BLOB_UPDATE_POLL_INTERVAL_SECS
Interval to check for the worker to check for updated blobs and refresh worker state accordingly. The default is 10 seconds- See Also:
-
STORM_DAEMON_METRICS_REPORTER_PLUGIN_LOCALE
Specify the Locale for daemon metrics reporter plugin. Use the specified IETF BCP 47 language tag string for a Locale. This config should have been placed in the DaemonConfig class since it is intended only for use by daemons. Keeping it here only for backwards compatibility.- See Also:
-
STORM_DAEMON_METRICS_REPORTER_PLUGIN_RATE_UNIT
Specify the rate unit in TimeUnit for daemon metrics reporter plugin. This config should have been placed in the DaemonConfig class since it is intended only for use by daemons. Keeping it here only for backwards compatibility.- See Also:
-
STORM_DAEMON_METRICS_REPORTER_PLUGIN_DURATION_UNIT
Specify the duration unit in TimeUnit for daemon metrics reporter plugin. This config should have been placed in the DaemonConfig class since it is intended only for use by daemons. Keeping it here only for backwards compatibility.- See Also:
-
-
Constructor Details
-
Config
public Config()
-
-
Method Details
-
setClasspath
-
setEnvironment
-
setDebug
-
setTopologyVersion
-
setNumWorkers
-
setNumAckers
-
setNumEventLoggers
-
setMessageTimeoutSecs
-
registerSerialization
-
registerSerialization
-
registerEventLogger
-
registerEventLogger
public static void registerEventLogger(Map<String, Object> conf, Class<? extends IEventLogger> klass) -
registerMetricsConsumer
-
registerMetricsConsumer
-
registerMetricsConsumer
-
registerDecorator
public static void registerDecorator(Map<String, Object> conf, Class<? extends IKryoDecorator> klass) -
setKryoFactory
-
setSkipMissingKryoRegistrations
-
setMaxTaskParallelism
-
setMaxSpoutPending
-
setStatsSampleRate
-
setFallBackOnJavaSerialization
-
setClasspath
-
setEnvironment
-
setDebug
public void setDebug(boolean isOn) -
setTopologyVersion
-
setNumWorkers
public void setNumWorkers(int workers) -
setNumAckers
public void setNumAckers(int numExecutors) -
setNumEventLoggers
public void setNumEventLoggers(int numExecutors) -
setMessageTimeoutSecs
public void setMessageTimeoutSecs(int secs) -
registerSerialization
-
registerSerialization
-
registerEventLogger
-
registerEventLogger
-
registerMetricsConsumer
-
registerMetricsConsumer
-
registerMetricsConsumer
-
registerDecorator
-
setKryoFactory
-
setSkipMissingKryoRegistrations
public void setSkipMissingKryoRegistrations(boolean skip) -
setMaxTaskParallelism
public void setMaxTaskParallelism(int max) -
setMaxSpoutPending
public void setMaxSpoutPending(int max) -
setStatsSampleRate
public void setStatsSampleRate(double rate) -
setFallBackOnJavaSerialization
public void setFallBackOnJavaSerialization(boolean fallback) -
setTopologyWorkerMaxHeapSize
Set the max heap size allow per worker for this topology.- Parameters:
size
- the maximum heap size for a worker.
-
setTopologyComponentWorkerConstraints
Declares executors of component1 cannot be on the same worker as executors of component2. This function is additive. Thus a user can setTopologyComponentWorkerConstraints("A", "B") and then setTopologyComponentWorkerConstraints("B", "C") Which means executors form component A cannot be on the same worker with executors of component B and executors of Component B cannot be on workers with executors of component C- Parameters:
component1
- a component that should not coexist with component2component2
- a component that should not coexist with component1
-
setTopologyConstraintsMaxStateSearch
public void setTopologyConstraintsMaxStateSearch(int numStates) Sets the maximum number of states that will be searched in the constraint solver strategy.- Parameters:
numStates
- maximum number of stats to search.
-
setTopologyPriority
public void setTopologyPriority(int priority) Set the priority for a topology. -
setTopologyStrategy
-
getBlobstoreHDFSPrincipal
Deprecated.- Throws:
UnknownHostException
-
getHdfsPrincipal
Get the hostname substituted hdfs principal.- Parameters:
conf
- the storm Configuration- Returns:
- the principal
- Throws:
UnknownHostException
- on UnknowHostException
-
getHdfsKeytab
Get the hdfs keytab.- Parameters:
conf
- the storm Configuration- Returns:
- the keytab
-