org.apache.cassandra.service
Interface StorageServiceMBean

All Known Implementing Classes:
StorageService

public interface StorageServiceMBean


Method Summary
 void bulkLoad(java.lang.String directory)
           
 void clearSnapshot(java.lang.String tag, java.lang.String... tableNames)
          Remove the snapshot with the given name from the given tables.
 void decommission()
          transfer this node's data to other machines and remove it from service.
 java.util.List<java.lang.String> describeRingJMX(java.lang.String keyspace)
          The same as describeRing(String) but converts TokenRange to the String for JMX compatibility
 void disableScheduledRangeXfers()
          Disable processing of queued range transfers.
 void drain()
          makes node unavailable for writes, flushes memtables and replays commitlog.
 java.util.Map<java.net.InetAddress,java.lang.Float> effectiveOwnership(java.lang.String keyspace)
          Effective ownership is % of the data each node owns given the keyspace we calculate the percentage using replication factor.
 void enableScheduledRangeXfers()
          Begin processing of queued range transfers.
 void forceRemoveCompletion()
          Force a remove operation to finish.
 void forceTableCleanup(java.lang.String tableName, java.lang.String... columnFamilies)
          Trigger a cleanup of keys on a single keyspace
 void forceTableCompaction(java.lang.String tableName, java.lang.String... columnFamilies)
          Forces major compaction of a single keyspace
 void forceTableFlush(java.lang.String tableName, java.lang.String... columnFamilies)
          Flush all memtables for the given column families, or all columnfamilies for the given table if none are explicitly listed.
 void forceTableRepair(java.lang.String tableName, boolean isSequential, boolean isLocal, java.lang.String... columnFamilies)
          Triggers proactive repair for given column families, or all columnfamilies for the given table if none are explicitly listed.
 void forceTableRepairPrimaryRange(java.lang.String tableName, boolean isSequential, boolean isLocal, java.lang.String... columnFamilies)
          Triggers proactive repair but only for the node primary range.
 void forceTableRepairRange(java.lang.String beginToken, java.lang.String endToken, java.lang.String tableName, boolean isSequential, boolean isLocal, java.lang.String... columnFamilies)
          Perform repair of a specific range.
 void forceTerminateAllRepairSessions()
           
 java.lang.String[] getAllDataFileLocations()
          Get the list of all data file locations from conf
 java.lang.String getCommitLogLocation()
          Get location of the commit log
 int getCompactionThroughputMbPerSec()
           
 int getCurrentGenerationNumber()
          Return the generation value for this node.
 java.lang.String getDrainProgress()
          get the progress of a drain operation
 int getExceptionCount()
           
 java.util.Map<java.lang.String,java.lang.String> getHostIdMap()
          Retrieve the mapping of endpoint to host ID
 java.util.List<java.lang.String> getJoiningNodes()
          Retrieve the list of nodes currently bootstrapping into the ring.
 java.util.List<java.lang.String> getKeyspaces()
           
 java.util.List<java.lang.String> getLeavingNodes()
          Retrieve the list of nodes currently leaving the ring.
 java.util.List<java.lang.String> getLiveNodes()
          Retrieve the list of live nodes in the cluster, where "liveness" is determined by the failure detector of the node being queried.
 double getLoad()
          Deprecated. 
 java.util.Map<java.lang.String,java.lang.String> getLoadMap()
          Human-readable load value.
 java.lang.String getLoadString()
          Human-readable load value
 java.lang.String getLocalHostId()
          Retrieve this hosts unique ID
 java.util.List<java.lang.String> getMovingNodes()
          Retrieve the list of nodes currently moving in the ring.
 java.util.List<java.net.InetAddress> getNaturalEndpoints(java.lang.String table, java.nio.ByteBuffer key)
           
 java.util.List<java.net.InetAddress> getNaturalEndpoints(java.lang.String table, java.lang.String cf, java.lang.String key)
          This method returns the N endpoints that are responsible for storing the specified key i.e for replication.
 java.lang.String getOperationMode()
          get the operational mode (leaving, joining, normal, decommissioned, client)
 java.util.Map<java.net.InetAddress,java.lang.Float> getOwnership()
          given a list of tokens (representing the nodes in the cluster), returns a mapping from "token -> %age of cluster owned by that token"
 java.util.Map<java.util.List<java.lang.String>,java.util.List<java.lang.String>> getPendingRangeToEndpointMap(java.lang.String keyspace)
          Retrieve a map of pending ranges to endpoints that describe the ring topology
 java.util.List<java.lang.String> getPrimaryRange()
          Returns the local node's primary range.
 java.util.Map<java.util.List<java.lang.String>,java.util.List<java.lang.String>> getRangeToEndpointMap(java.lang.String keyspace)
          Retrieve a map of range to end points that describe the ring topology of a Cassandra cluster.
 java.util.Map<java.util.List<java.lang.String>,java.util.List<java.lang.String>> getRangeToRpcaddressMap(java.lang.String keyspace)
          Retrieve a map of range to rpc addresses that describe the ring topology of a Cassandra cluster.
 java.lang.String getReleaseVersion()
          Fetch a string representation of the Cassandra version.
 java.lang.String getRemovalStatus()
          Get the status of a token removal.
 java.lang.String getSavedCachesLocation()
          Get location of the saved caches dir
 java.lang.String getSchemaVersion()
          Fetch a string representation of the current Schema version.
 int getStreamThroughputMbPerSec()
           
 java.util.List<java.lang.String> getTokens()
          Fetch string representations of the tokens for this node.
 java.util.List<java.lang.String> getTokens(java.lang.String endpoint)
          Fetch string representations of the tokens for a specified node.
 java.util.Map<java.lang.String,java.lang.String> getTokenToEndpointMap()
          Retrieve a map of tokens to endpoints, including the bootstrapping ones.
 double getTracingProbability()
          Returns the configured tracing probability.
 java.util.List<java.lang.String> getUnreachableNodes()
          Retrieve the list of unreachable nodes in the cluster, as determined by this node's failure detector.
 boolean isIncrementalBackupsEnabled()
           
 boolean isInitialized()
           
 boolean isJoined()
           
 boolean isNativeTransportRunning()
           
 boolean isRPCServerRunning()
           
 void joinRing()
           
 void loadNewSSTables(java.lang.String ksName, java.lang.String cfName)
          Load new SSTables to the given keyspace/columnFamily
 void move(java.lang.String newToken)
           
 void rebuild(java.lang.String sourceDc)
          Initiate a process of streaming data for which we are responsible from other nodes.
 void rebuildSecondaryIndex(java.lang.String ksName, java.lang.String cfName, java.lang.String... idxNames)
          rebuild the specified indexes
 void relocate(java.util.Collection<java.lang.String> srcTokens)
           
 void removeNode(java.lang.String token)
          removeToken removes token (and all data associated with enpoint that had it) from the ring
 void rescheduleFailedDeletions()
           
 void resetLocalSchema()
           
 java.util.List<java.lang.String> sampleKeyRange()
          Return a List of Tokens representing a sample of keys across all ColumnFamilyStores.
 void scrub(java.lang.String tableName, java.lang.String... columnFamilies)
          Scrub (deserialize + reserialize at the latest version, skipping bad rows if any) the given keyspace.
 void setCompactionThroughputMbPerSec(int value)
           
 void setIncrementalBackupsEnabled(boolean value)
           
 void setLog4jLevel(java.lang.String classQualifier, java.lang.String level)
          set the logging level at runtime
 void setStreamThroughputMbPerSec(int value)
           
 void setTraceProbability(double probability)
          Enables/Disables tracing for the whole system.
 void startGossiping()
           
 void startNativeTransport()
           
 void startRPCServer()
           
 void stopGossiping()
           
 void stopNativeTransport()
           
 void stopRPCServer()
           
 void takeColumnFamilySnapshot(java.lang.String tableName, java.lang.String columnFamilyName, java.lang.String tag)
          Takes the snapshot of a specific column family.
 void takeSnapshot(java.lang.String tag, java.lang.String... tableNames)
          Takes the snapshot for the given tables.
 void truncate(java.lang.String keyspace, java.lang.String columnFamily)
          Truncates (deletes) the given columnFamily from the provided keyspace.
 void updateSnitch(java.lang.String epSnitchClassName, java.lang.Boolean dynamic, java.lang.Integer dynamicUpdateInterval, java.lang.Integer dynamicResetInterval, java.lang.Double dynamicBadnessThreshold)
          Change endpointsnitch class and dynamic-ness (and dynamic attributes) at runtime
 void upgradeSSTables(java.lang.String tableName, java.lang.String... columnFamilies)
          Rewrite all sstables to the latest version.
 

Method Detail

getLiveNodes

java.util.List<java.lang.String> getLiveNodes()
Retrieve the list of live nodes in the cluster, where "liveness" is determined by the failure detector of the node being queried.

Returns:
set of IP addresses, as Strings

getUnreachableNodes

java.util.List<java.lang.String> getUnreachableNodes()
Retrieve the list of unreachable nodes in the cluster, as determined by this node's failure detector.

Returns:
set of IP addresses, as Strings

getJoiningNodes

java.util.List<java.lang.String> getJoiningNodes()
Retrieve the list of nodes currently bootstrapping into the ring.

Returns:
set of IP addresses, as Strings

getLeavingNodes

java.util.List<java.lang.String> getLeavingNodes()
Retrieve the list of nodes currently leaving the ring.

Returns:
set of IP addresses, as Strings

getMovingNodes

java.util.List<java.lang.String> getMovingNodes()
Retrieve the list of nodes currently moving in the ring.

Returns:
set of IP addresses, as Strings

getTokens

java.util.List<java.lang.String> getTokens()
Fetch string representations of the tokens for this node.

Returns:
a collection of tokens formatted as strings

getTokens

java.util.List<java.lang.String> getTokens(java.lang.String endpoint)
                                           throws java.net.UnknownHostException
Fetch string representations of the tokens for a specified node.

Parameters:
endpoint - string representation of an node
Returns:
a collection of tokens formatted as strings
Throws:
java.net.UnknownHostException

getReleaseVersion

java.lang.String getReleaseVersion()
Fetch a string representation of the Cassandra version.

Returns:
A string representation of the Cassandra version.

getSchemaVersion

java.lang.String getSchemaVersion()
Fetch a string representation of the current Schema version.

Returns:
A string representation of the Schema version.

getAllDataFileLocations

java.lang.String[] getAllDataFileLocations()
Get the list of all data file locations from conf

Returns:
String array of all locations

getCommitLogLocation

java.lang.String getCommitLogLocation()
Get location of the commit log

Returns:
a string path

getSavedCachesLocation

java.lang.String getSavedCachesLocation()
Get location of the saved caches dir

Returns:
a string path

getRangeToEndpointMap

java.util.Map<java.util.List<java.lang.String>,java.util.List<java.lang.String>> getRangeToEndpointMap(java.lang.String keyspace)
Retrieve a map of range to end points that describe the ring topology of a Cassandra cluster.

Returns:
mapping of ranges to end points

getRangeToRpcaddressMap

java.util.Map<java.util.List<java.lang.String>,java.util.List<java.lang.String>> getRangeToRpcaddressMap(java.lang.String keyspace)
Retrieve a map of range to rpc addresses that describe the ring topology of a Cassandra cluster.

Returns:
mapping of ranges to rpc addresses

describeRingJMX

java.util.List<java.lang.String> describeRingJMX(java.lang.String keyspace)
                                                 throws java.io.IOException
The same as describeRing(String) but converts TokenRange to the String for JMX compatibility

Parameters:
keyspace - The keyspace to fetch information about
Returns:
a List of TokenRange(s) converted to String for the given keyspace
Throws:
java.io.IOException

getPrimaryRange

java.util.List<java.lang.String> getPrimaryRange()
Returns the local node's primary range.


getPendingRangeToEndpointMap

java.util.Map<java.util.List<java.lang.String>,java.util.List<java.lang.String>> getPendingRangeToEndpointMap(java.lang.String keyspace)
Retrieve a map of pending ranges to endpoints that describe the ring topology

Parameters:
keyspace - the keyspace to get the pending range map for.
Returns:
a map of pending ranges to endpoints

getTokenToEndpointMap

java.util.Map<java.lang.String,java.lang.String> getTokenToEndpointMap()
Retrieve a map of tokens to endpoints, including the bootstrapping ones.

Returns:
a map of tokens to endpoints in ascending order

getLocalHostId

java.lang.String getLocalHostId()
Retrieve this hosts unique ID


getHostIdMap

java.util.Map<java.lang.String,java.lang.String> getHostIdMap()
Retrieve the mapping of endpoint to host ID


getLoad

@Deprecated
double getLoad()
Deprecated. 

Numeric load value.

See Also:
StorageMetrics.load

getLoadString

java.lang.String getLoadString()
Human-readable load value


getLoadMap

java.util.Map<java.lang.String,java.lang.String> getLoadMap()
Human-readable load value. Keys are IP addresses.


getCurrentGenerationNumber

int getCurrentGenerationNumber()
Return the generation value for this node.

Returns:
generation number

getNaturalEndpoints

java.util.List<java.net.InetAddress> getNaturalEndpoints(java.lang.String table,
                                                         java.lang.String cf,
                                                         java.lang.String key)
This method returns the N endpoints that are responsible for storing the specified key i.e for replication.

Parameters:
table - keyspace name also known as table
cf - Column family name
key - - key for which we need to find the endpoint return value - the endpoint responsible for this key

getNaturalEndpoints

java.util.List<java.net.InetAddress> getNaturalEndpoints(java.lang.String table,
                                                         java.nio.ByteBuffer key)

takeSnapshot

void takeSnapshot(java.lang.String tag,
                  java.lang.String... tableNames)
                  throws java.io.IOException
Takes the snapshot for the given tables. A snapshot name must be specified.

Parameters:
tag - the tag given to the snapshot; may not be null or empty
tableNames - the name of the tables to snapshot; empty means "all."
Throws:
java.io.IOException

takeColumnFamilySnapshot

void takeColumnFamilySnapshot(java.lang.String tableName,
                              java.lang.String columnFamilyName,
                              java.lang.String tag)
                              throws java.io.IOException
Takes the snapshot of a specific column family. A snapshot name must be specified.

Parameters:
tableName - the keyspace which holds the specified column family
columnFamilyName - the column family to snapshot
tag - the tag given to the snapshot; may not be null or empty
Throws:
java.io.IOException

clearSnapshot

void clearSnapshot(java.lang.String tag,
                   java.lang.String... tableNames)
                   throws java.io.IOException
Remove the snapshot with the given name from the given tables. If no tag is specified we will remove all snapshots.

Throws:
java.io.IOException

forceTableCompaction

void forceTableCompaction(java.lang.String tableName,
                          java.lang.String... columnFamilies)
                          throws java.io.IOException,
                                 java.util.concurrent.ExecutionException,
                                 java.lang.InterruptedException
Forces major compaction of a single keyspace

Throws:
java.io.IOException
java.util.concurrent.ExecutionException
java.lang.InterruptedException

forceTableCleanup

void forceTableCleanup(java.lang.String tableName,
                       java.lang.String... columnFamilies)
                       throws java.io.IOException,
                              java.util.concurrent.ExecutionException,
                              java.lang.InterruptedException
Trigger a cleanup of keys on a single keyspace

Throws:
java.io.IOException
java.util.concurrent.ExecutionException
java.lang.InterruptedException

scrub

void scrub(java.lang.String tableName,
           java.lang.String... columnFamilies)
           throws java.io.IOException,
                  java.util.concurrent.ExecutionException,
                  java.lang.InterruptedException
Scrub (deserialize + reserialize at the latest version, skipping bad rows if any) the given keyspace. If columnFamilies array is empty, all CFs are scrubbed. Scrubbed CFs will be snapshotted first.

Throws:
java.io.IOException
java.util.concurrent.ExecutionException
java.lang.InterruptedException

upgradeSSTables

void upgradeSSTables(java.lang.String tableName,
                     java.lang.String... columnFamilies)
                     throws java.io.IOException,
                            java.util.concurrent.ExecutionException,
                            java.lang.InterruptedException
Rewrite all sstables to the latest version. Unlike scrub, it doesn't skip bad rows and do not snapshot sstables first.

Throws:
java.io.IOException
java.util.concurrent.ExecutionException
java.lang.InterruptedException

forceTableFlush

void forceTableFlush(java.lang.String tableName,
                     java.lang.String... columnFamilies)
                     throws java.io.IOException,
                            java.util.concurrent.ExecutionException,
                            java.lang.InterruptedException
Flush all memtables for the given column families, or all columnfamilies for the given table if none are explicitly listed.

Parameters:
tableName -
columnFamilies -
Throws:
java.io.IOException
java.util.concurrent.ExecutionException
java.lang.InterruptedException

forceTableRepair

void forceTableRepair(java.lang.String tableName,
                      boolean isSequential,
                      boolean isLocal,
                      java.lang.String... columnFamilies)
                      throws java.io.IOException
Triggers proactive repair for given column families, or all columnfamilies for the given table if none are explicitly listed.

Parameters:
tableName -
columnFamilies -
Throws:
java.io.IOException

forceTableRepairPrimaryRange

void forceTableRepairPrimaryRange(java.lang.String tableName,
                                  boolean isSequential,
                                  boolean isLocal,
                                  java.lang.String... columnFamilies)
                                  throws java.io.IOException
Triggers proactive repair but only for the node primary range.

Throws:
java.io.IOException

forceTableRepairRange

void forceTableRepairRange(java.lang.String beginToken,
                           java.lang.String endToken,
                           java.lang.String tableName,
                           boolean isSequential,
                           boolean isLocal,
                           java.lang.String... columnFamilies)
                           throws java.io.IOException
Perform repair of a specific range. This allows incremental repair to be performed by having an external controller submitting repair jobs. Note that the provided range much be a subset of one of the node local range.

Throws:
java.io.IOException

forceTerminateAllRepairSessions

void forceTerminateAllRepairSessions()

decommission

void decommission()
                  throws java.lang.InterruptedException
transfer this node's data to other machines and remove it from service.

Throws:
java.lang.InterruptedException

move

void move(java.lang.String newToken)
          throws java.io.IOException
Parameters:
newToken - token to move this node to. This node will unload its data onto its neighbors, and bootstrap to the new token.
Throws:
java.io.IOException

relocate

void relocate(java.util.Collection<java.lang.String> srcTokens)
              throws java.io.IOException
Parameters:
srcTokens - tokens to move to this node
Throws:
java.io.IOException

removeNode

void removeNode(java.lang.String token)
removeToken removes token (and all data associated with enpoint that had it) from the ring


getRemovalStatus

java.lang.String getRemovalStatus()
Get the status of a token removal.


forceRemoveCompletion

void forceRemoveCompletion()
Force a remove operation to finish.


setLog4jLevel

void setLog4jLevel(java.lang.String classQualifier,
                   java.lang.String level)
set the logging level at runtime


getOperationMode

java.lang.String getOperationMode()
get the operational mode (leaving, joining, normal, decommissioned, client)


getDrainProgress

java.lang.String getDrainProgress()
get the progress of a drain operation


drain

void drain()
           throws java.io.IOException,
                  java.lang.InterruptedException,
                  java.util.concurrent.ExecutionException
makes node unavailable for writes, flushes memtables and replays commitlog.

Throws:
java.io.IOException
java.lang.InterruptedException
java.util.concurrent.ExecutionException

truncate

void truncate(java.lang.String keyspace,
              java.lang.String columnFamily)
              throws java.util.concurrent.TimeoutException,
                     java.io.IOException
Truncates (deletes) the given columnFamily from the provided keyspace. Calling truncate results in actual deletion of all data in the cluster under the given columnFamily and it will fail unless all hosts are up. All data in the given column family will be deleted, but its definition will not be affected.

Parameters:
keyspace - The keyspace to delete from
columnFamily - The column family to delete data from.
Throws:
java.util.concurrent.TimeoutException
java.io.IOException

getOwnership

java.util.Map<java.net.InetAddress,java.lang.Float> getOwnership()
given a list of tokens (representing the nodes in the cluster), returns a mapping from "token -> %age of cluster owned by that token"


effectiveOwnership

java.util.Map<java.net.InetAddress,java.lang.Float> effectiveOwnership(java.lang.String keyspace)
                                                                       throws java.lang.IllegalStateException
Effective ownership is % of the data each node owns given the keyspace we calculate the percentage using replication factor. If Keyspace == null, this method will try to verify if all the keyspaces in the cluster have the same replication strategies and if yes then we will use the first else a empty Map is returned.

Throws:
java.lang.IllegalStateException

getKeyspaces

java.util.List<java.lang.String> getKeyspaces()

updateSnitch

void updateSnitch(java.lang.String epSnitchClassName,
                  java.lang.Boolean dynamic,
                  java.lang.Integer dynamicUpdateInterval,
                  java.lang.Integer dynamicResetInterval,
                  java.lang.Double dynamicBadnessThreshold)
                  throws java.lang.ClassNotFoundException
Change endpointsnitch class and dynamic-ness (and dynamic attributes) at runtime

Parameters:
epSnitchClassName - the canonical path name for a class implementing IEndpointSnitch
dynamic - boolean that decides whether dynamicsnitch is used or not
dynamicUpdateInterval - integer, in ms (default 100)
dynamicResetInterval - integer, in ms (default 600,000)
dynamicBadnessThreshold - double, (default 0.0)
Throws:
java.lang.ClassNotFoundException

stopGossiping

void stopGossiping()

startGossiping

void startGossiping()

isInitialized

boolean isInitialized()

stopRPCServer

void stopRPCServer()

startRPCServer

void startRPCServer()

isRPCServerRunning

boolean isRPCServerRunning()

stopNativeTransport

void stopNativeTransport()

startNativeTransport

void startNativeTransport()

isNativeTransportRunning

boolean isNativeTransportRunning()

joinRing

void joinRing()
              throws java.io.IOException
Throws:
java.io.IOException

isJoined

boolean isJoined()

getExceptionCount

int getExceptionCount()

setStreamThroughputMbPerSec

void setStreamThroughputMbPerSec(int value)

getStreamThroughputMbPerSec

int getStreamThroughputMbPerSec()

getCompactionThroughputMbPerSec

int getCompactionThroughputMbPerSec()

setCompactionThroughputMbPerSec

void setCompactionThroughputMbPerSec(int value)

isIncrementalBackupsEnabled

boolean isIncrementalBackupsEnabled()

setIncrementalBackupsEnabled

void setIncrementalBackupsEnabled(boolean value)

rebuild

void rebuild(java.lang.String sourceDc)
Initiate a process of streaming data for which we are responsible from other nodes. It is similar to bootstrap except meant to be used on a node which is already in the cluster (typically containing no data) as an alternative to running repair.

Parameters:
sourceDc - Name of DC from which to select sources for streaming or null to pick any node

bulkLoad

void bulkLoad(java.lang.String directory)

rescheduleFailedDeletions

void rescheduleFailedDeletions()

loadNewSSTables

void loadNewSSTables(java.lang.String ksName,
                     java.lang.String cfName)
Load new SSTables to the given keyspace/columnFamily

Parameters:
ksName - The parent keyspace name
cfName - The ColumnFamily name where SSTables belong

sampleKeyRange

java.util.List<java.lang.String> sampleKeyRange()
Return a List of Tokens representing a sample of keys across all ColumnFamilyStores. Note: this should be left as an operation, not an attribute (methods starting with "get") to avoid sending potentially multiple MB of data when accessing this mbean by default. See CASSANDRA-4452.

Returns:
set of Tokens as Strings

rebuildSecondaryIndex

void rebuildSecondaryIndex(java.lang.String ksName,
                           java.lang.String cfName,
                           java.lang.String... idxNames)
rebuild the specified indexes


resetLocalSchema

void resetLocalSchema()
                      throws java.io.IOException
Throws:
java.io.IOException

setTraceProbability

void setTraceProbability(double probability)
Enables/Disables tracing for the whole system. Only thrift requests can start tracing currently.

Parameters:
probability - ]0,1[ will enable tracing on a partial number of requests with the provided probability. 0 will disable tracing and 1 will enable tracing for all requests (which mich severely cripple the system)

getTracingProbability

double getTracingProbability()
Returns the configured tracing probability.


enableScheduledRangeXfers

void enableScheduledRangeXfers()
Begin processing of queued range transfers.


disableScheduledRangeXfers

void disableScheduledRangeXfers()
Disable processing of queued range transfers.



Copyright © 2012 The Apache Software Foundation