The cassandra.yaml configuration file
The cassandra.yaml file is the main configuration file for Cassandra.
The cassandra.yaml file is the main configuration file for Cassandra.
- Cassandra package installations: /etc/cassandra
- Cassandra tarball installations: install_location/conf
The configuration properties are grouped into the following sections:
- Quick start
                                                  The minimal properties needed for configuring a cluster. 
- Commonly
                                                usedProperties most frequently used when configuring Cassandra. 
- Performance tuningTuning performance and system resource utilization, including commit log, compaction, memory, disk I/O, CPU, reads, and writes. 
- AdvancedProperties for advanced users or properties that are less commonly used. 
- SecurityServer and client security settings. 
Quick start properties
The minimal properties needed for configuring a cluster.
Related information: Initializing a multiple node cluster (single datacenter) and Initializing a multiple node cluster (multiple datacenters).
- cluster_name
- (Default: Test Cluster) The name of the cluster. This setting prevents nodes in one logical cluster from joining another. All nodes in a cluster must have the same value.
- listen_address
- (Default: localhost) The IP address or hostname
                                                  that Cassandra binds to for connecting this node
                                                  to other nodes. Set this parameter or listen_interface, not both. Correct
                                                settings for various use cases:- Single-node installations: do one of
                                                  the following:- Comment this property out. If the node is properly configured (host name, name resolution, and so on.), Cassandra uses InetAddress.getLocalHost() to get the local address from the system.
- Leave set to the default,
                                                  localhost.
 
- Node in a multi-node installations: set this property to the node's IP address or hostname, or set listen_interface.
- Node in a multi-network or multi-Datacenter
                                                  installation, within an EC2 environment that
                                                  supports automatic switching between public and
                                                  private interfaces: set
                                                  listen_addressto the node's IP address or hostname, or set listen_interface.
- Node with two physical network interfaces
                                                  in a multi-datacenter installation or a Cassandra
                                                  cluster deployed across multiple Amazon EC2
                                                  regions using the
                                                  Ec2MultiRegionSnitch:- Set listen_addressto this node's private IP or hostname, or set listen_interface (for communication within the local datacenter).
- Set broadcast_address to the second IP or hostname (for communication between datacenters).
- Set listen_on_broadcast_address to
                                                  true.
- If this node is a seed node, add the node's public IP address or hostname to the seeds list.
 
- Set 
- Open the storage_port or ssl_storage_port on the public IP firewall.
 Warning:- 
                                                  Never set listen_addressto 0.0.0.0. It is always wrong.
- 
                                                  Do not set values for both listen_address and listen_interfaceon the same node.
 
- Single-node installations: do one of
                                                  the following:
- listen_interface
- (Default: eth0)note The interface that
                                                Cassandra binds to for connecting to other Cassandra
                                                nodes. Interfaces must correspond to a single
                                                address — IP aliasing is not supported. Do not set
                                                values for both listen_address and
                                                  listen_interfaceon the same node.
- listen_interface_prefer_ipv6
- (Default: false) If an interface has an ipv4 and an ipv6 address, Cassandra uses the first ipv4 address by default. Set this property to true to configure Cassandra to use the first ipv6 address.
Default directories
If you have changed any of the default directories during installation, set these properties to the new locations. Make sure you have root access.
- cdc_raw_directory
- The directory where the CDC log is stored. Default
                                                locations:- Package installations: /var/lib/cassandra/cdc_raw
- Tarball installations: install_location/data/cdc_raw
 The directory where Change Data Capture logs are stored. 
- commitlog_directory
- The directory where the commit log is stored. Default
                                                  locations:- Package installations: /var/lib/cassandra/commitlog
- Tarball installations: install_location/data/commitlog
 For optimal write performance, place the commit log be on a separate disk partition, or (ideally) a separate physical device from the data file directories. Because the commit log is append only, an HDD is acceptable for this purpose. 
- data_file_directories
- The directory location where table data is stored (in
                                                SSTables). Cassandra distributes data evenly across
                                                the location, subject to the granularity of the
                                                configured compaction strategy. Default locations:
                                                  - Package installations: /var/lib/cassandra/data
- Tarball installations: install_location/data/data
 As a production best practice, use RAID 0 and SSDs. 
Commonly used properties
Properties most frequently used when configuring Cassandra.
Before starting a node for the first time, you should carefully evaluate your requirements.
Common initialization properties
- commit_failure_policy
- (Default: stop) Policy for commit disk failures:- dieShut down gossip and Thrift and kill the JVM, so the node can be replaced. 
- stopShut down gossip and Thrift, leaving the node effectively dead, available for inspection using JMX. 
- stop_commitShut down the commit log, letting writes collect but continuing to service reads (as in pre-2.0.5 Cassandra). 
- ignoreIgnore fatal errors and let the batches fail. 
 
- die
- disk_optimization_strategy
- (Default: ssd) The strategy for optimizing disk reads. Possible values: ssd or spinning.
- disk_failure_policy
- (Default: stop) Sets how Cassandra
                                                responds to disk failure. Recommend settings:
                                                  stop or
                                                  best_effort. Valid values:- dieShut down gossip and Thrift and kill the JVM for any file system errors or single SSTable errors, so the node can be replaced. 
- stop_paranoidShut down gossip and Thrift even for single SSTable errors. 
- stopShut down gossip and Thrift, leaving the node effectively dead, but available for inspection using JMX. 
- best_effortStop using the failed disk and respond to requests based on the remaining available SSTables. This allows obsolete data at consistency level of ONE. 
- ignore
                                                  Ignore fatal errors and lets the requests fail; all file system errors are logged but otherwise ignored. Cassandra acts as in versions prior to 1.2. 
 Related information: Handling Disk Failures In Cassandra 1.2 blog and Recovering from a single disk failure using JBOD. 
- die
- endpoint_snitch
- (Default:
                                                  org.apache.cassandra.locator.SimpleSnitch)
                                                Set to a class that implements the
                                                  IEndpointSnitch interface.
                                                Cassandra uses the snitch to locate nodes and route requests.- SimpleSnitchUse for single-datacenter deployment or single-zone deployment in public clouds. Does not recognize datacenter or rack information. Treats strategy order as proximity, which can improve cache locality when you disable read repair. 
- GossipingPropertyFileSnitch
                                                  Recommended for production. Reads rack and datacenter for the local node in cassandra-rackdc.properties file and propagates these values to other nodes via gossip. For migration from the PropertyFileSnitch, uses the cassandra-topology.properties file if it is present. The location of the cassandra-rackdc.properties file depends on the type of installation:Package installations /etc/cassandra/cassandra-rackdc.properties Tarball installations install_location/conf/cassandra-rackdc.properties The location of the cassandra-topology.properties file depends on the type of installation:Package installations /etc/cassandra/cassandra-topology.properties Tarball installations install_location/conf/cassandra-topology.properties 
- PropertyFileSnitchDetermines proximity by rack and datacenter, which are explicitly configured in cassandra-topology.properties file. 
- Ec2SnitchFor EC2 deployments in a single region. Loads region and availability zone information from the Amazon EC2 API. The region is treated as the datacenter and the availability zone as the rack and uses only private IP addresses. For this reason, it does not work across multiple regions. 
- Ec2MultiRegionSnitchUses the public IP as the broadcast_address to allow cross-region connectivity. This means you must also set seed addresses to the public IP and open the storage_port or ssl_storage_port on the public IP firewall. For intra-region traffic, Cassandra switches to the private IP after establishing a connection. 
- RackInferringSnitch: Proximity is determined by rack and datacenter, which are assumed to correspond to the 3rd and 2nd octet of each node's IP address, respectively. Best used as an example for writing a custom snitch class (unless this happens to match your deployment conventions). 
- GoogleCloudSnitch:Use for Cassandra deployments on Google Cloud Platform across one or more regions. The region is treated as a datacenter and the availability zones are treated as racks within the datacenter. All communication occurs over private IP addresses within the same logical network. 
- CloudstackSnitchUse the CloudstackSnitch for Apache Cloudstack environments. 
 Related information: Snitches 
- SimpleSnitch
- rpc_address
- (Default: localhost) The listen address
                                                for client connections (Thrift RPC service and
                                                native transport). Valid values:- unset: Resolves the address using the configured hostname configuration of the node. If left unset, the hostname resolves to the IP address of this node using /etc/hostname, /etc/hosts, or DNS. 
- 
                                                  0.0.0.0: Listens on all configured interfaces. You must set the broadcast_rpc_address to a value other than 0.0.0.0. 
- IP address
- hostname
 Related information: Network 
- unset: 
- rpc_interface
- (Default: eth1)note The listen address for client connections. Interface must correspond to a single address, IP aliasing is not supported. See rpc_address.
- rpc_interface_prefer_ipv6
- (Default: false) If an interface has an ipv4 and an ipv6 address, Cassandra uses the first ipv4 address by default, i. If set to true, the first ipv6 address will be used.
- seed_provider
- The addresses of hosts designated as contact points in
                                                the cluster. A joining node contacts one of the
                                                nodes in the -seeds list to
                                                learn the topology of the ring. - class_name (Default:
                                                  org.apache.cassandra.locator.SimpleSeedProvider)
                                                  The class within Cassandra that handles the seed logic. It can be customized, but this is typically not required. 
- - seeds (Default:
                                                  127.0.0.1) A comma-delimited list of IP addresses used by gossip for bootstrapping new nodes joining a cluster. If your cluster includes multiple nodes, you must change the list from the default value to the IP address of one of the nodes.Attention: In multiple data-center clusters, include at least one node from each datacenter (replication group) in the seed list. Designating more than a single seed node per datacenter is recommended for fault tolerance. Otherwise, gossip has to communicate with another datacenter when bootstrapping a node.Making every node a seed node is not recommended because of increased maintenance and reduced gossip performance. Gossip optimization is not critical, but it is recommended to use a small seed list (approximately three nodes per datacenter). 
 Related information: Initializing a multiple node cluster (single datacenter) and Initializing a multiple node cluster (multiple datacenters). 
- class_name (Default:
                                                  org.apache.cassandra.locator.SimpleSeedProvider)
                                                  
- enable_user_defined_functions
- (Default: false) User defined functions (UDFs) present a security risk, since they are executed on the server side. In Cassandra 3.0 and later, UDFs are executed in a sandbox to contain the execution of malicious code. They are disabled by default.
- enable_scripted_user_defined_functions
- (Default: false)  Java UDFs are always
                                                enabled, if
                                                  enable_user_defined_functionsis true. Enable this option to use UDFs withlanguage javascriptor any custom JSR-223 provider. This option has no effect ifenable_user_defined_functionsis false.
Common compaction settings
- compaction_throughput_mb_per_sec
- (Default: 16) Throttles compaction to
                                                the specified Mb/second across the instance. The
                                                faster Cassandra inserts data, the faster the system
                                                must compact in order to keep the SSTable count
                                                down. The recommended value is 16 to 32 times the
                                                rate of write throughput (in Mb/second). Setting the
                                                value to 0 disables compaction throttling.
                                                  Related information: Configuring compaction 
- compaction_large_partition_warning_threshold_mb
- (Default: 100) Cassandra logs a warning when compacting partitions larger than the set value.
Common memtable settings
- memtable_heap_space_in_mb
- (Default: 1/4 of heap size)note
- memtable_offheap_space_in_mb
- (Default: 1/4 of heap size)note
Common disk settings
- concurrent_reads
- (Default: 32)note Workloads with more data
                                                than can fit in memory encounter a bottleneck in
                                                fetching data from disk during reads. Setting
                                                  concurrent_readsto (16 × number_of_drives) allows operations to queue low enough in the stack so that the OS and drives can reorder them. The default setting applies to both logical volume managed (LVM) and RAID drives.
- concurrent_writes
- (Default: 32)note Writes in Cassandra are rarely I/O bound, so the ideal number of concurrent writes depends on the number of CPU cores on the node. The recommended value is 8 × number_of_cpu_cores.
- concurrent_counter_writes
- (Default: 32)note Counter writes read the current values before incrementing and writing them back. The recommended value is (16 × number_of_drives) .
- concurrent_batchlog_writes
- (Default: 32) Limit on the number of
                                                concurrent batchlog writes, similar to
                                                  concurrent_writes.
- concurrent_materialized_view_writes
- (Default: 32) Limit on the number of concurrent materialized view writes. Set this to the lesser of concurrent reads or concurrent writes, because there is a read involved in each materialized view write.
Common automatic backup settings
- incremental_backups
- (Default: false) Backs up data updated
                                                since the last snapshot was taken. When enabled,
                                                Cassandra creates a hard link to each SSTable
                                                flushed or streamed locally in a
                                                  backups subdirectory of the
                                                keyspace data. Removing these links is the
                                                operator's responsibility. Related information: Enabling incremental backups 
- snapshot_before_compaction
- (Default: false) Enables or disables
                                                taking a snapshot before each compaction. A snapshot
                                                is useful to back up data when there is a data
                                                format change. Be careful using this option:
                                                Cassandra does not clean up older snapshots
                                                automatically. Related information: Configuring compaction 
Common fault detection setting
- phi_convict_threshold
- (Default: 8)note Adjusts the sensitivity of
                                                the failure detector on an exponential scale.
                                                Generally this setting does not need adjusting.
                                                  Related information: Failure detection and recovery 
Performance tuning properties
Tuning performance and system resource utilization, including commit log, compaction, memory, disk I/O, CPU, reads, and writes.
Commit log settings
- commitlog_sync
- (Default: periodic) The method that
                                                Cassandra uses to acknowledge writes in
                                                  milliseconds:- 
                                                  periodic: (Default:
                                                  10000 milliseconds [10 seconds])
                                                  With commitlog_sync_period_in_ms, controls how often the commit log is synchronized to disk. Periodic syncs are acknowledged immediately. 
- batch: (Default:
                                                  disabled)noteWith commitlog_sync_batch_window_in_ms (Default: 2 ms), controls how long Cassandra waits for other writes before performing a sync. When this method is enabled, Cassandra does not acknowledge writes until they are fsynced to disk. 
 
- 
                                                  periodic: (Default:
                                                  10000 milliseconds [10 seconds])
                                                  
- commitlog_segment_size_in_mb
- (Default: 32MB) The size of an
                                                individual commitlog file segment. A commitlog
                                                segment may be archived, deleted, or recycled after
                                                all its data has been flushed to SSTables. This data
                                                can potentially include commitlog segments from
                                                every table in the system. The default size is
                                                usually suitable for most commitlog archiving, but
                                                if you want a finer granularity, 8 or 16 MB is
                                                reasonable. By default, the max_mutation_size_in_kb is set to half of the commitlog_segment_size_in_kb.Related information: Commit log archive configuration 
- max_mutation_size_in_kb
- (Default: ½ of commitlog_segment_size_in_mb. If a mutation's size exceeds this value, the mutation is rejected. Before increasing the commitlog segment size of the commitlog segments, investigate why the mutations are larger than expected. Look for underlying issues with access patterns and data model, because increasing the commitlog segment size is a limited fix. Restriction:If you set max_mutation_size_in_kbexplicitly, then you must setcommitlog_segment_size_in_mbto at least twice the size of max_mutation_size_in_kb / 1024.For more information, see commitlog_segment_size_in_mb above. 
- commitlog_compression
- (Default: not enabled) The compressor to use if commit
                                                log is compressed. Valid values:
                                                  LZ4,SnappyorDeflate. If no value is set for this property, the commit log is written uncompressed.
- cdc_total_space_in_mb
- (Default: 4096MB and
                                                  1/8thof the total space of the
                                                drive where the cdc_raw_directoryresides.)note If space gets above this value, Cassandra will throwWriteTimeoutExceptionon Mutations including tables with CDC enabled. A CDCCompactor (a consumer) is responsible for parsing the raw CDC logs and deleting them when parsing is completed.
- cdc_free_space_check_interval_ms
- (Default: 250 ms)note When
                                                the cdc_rawlimit is hit and the CDCCompactor is either running behind or experiencing backpressure, this interval is checked to see if any new space for cdc-tracked tables has been made available.
- commitlog_total_space_in_mb
- (Default: 32MB for 32-bit JVMs,
                                                  8192MB for 64-bit JVMs)note Total space used for
                                                commit logs. If the total space used by all commit
                                                logs goes above this value, Cassandra rounds up to
                                                the next nearest segment multiple and flushes
                                                memtables to disk for the oldest commitlog segments,
                                                removing those log segments from the commit log.
                                                This reduces the amount of data to replay on
                                                start-up, and prevents infrequently-updated tables
                                                from keeping commitlog segments indefinitely. If the
                                                  commitlog_total_space_in_mbis small, the result is more flush activity on less-active tables.Related information: Configuring memtable thresholds 
Compaction settings
Related information: Configuring compaction
- concurrent_compactors
- (Default: Smaller of number of disks or number of cores,
                                                with a minimum of 2 and a maximum of 8 per CPU
                                                  core)noteThe number of concurrent
                                                compaction processes allowed to run simultaneously
                                                on a node, not including validation compactions
                                                  for
                                                  anti-entropy repair. Simultaneous
                                                compactions help preserve read performance in a
                                                mixed read-write workload by limiting the number of
                                                small SSTables that accumulate during a single
                                                long-running compaction. If your data directories
                                                are backed by SSDs,
                                                increase this value to the number of cores. If
                                                compaction running too slowly or too fast, adjust
                                                  compaction_throughput_mb_per_sec
                                                  first.Note: Increasing concurrent compactors leads to more use of available disk space for compaction, because concurrent compactions happen in parallel, especially for STCS. Ensure that adequate disk space is available before increasing this configuration.
- sstable_preemptive_open_interval_in_mb
- (Default: 50MB) The compaction process opens SSTables before they are completely written and uses them in place of the prior SSTables for any range previously written. This setting helps to smoothly transfer reads between the SSTables by reducing page cache churn and keeps hot rows hot.
Memtable settings
- memtable_allocation_type
- (Default: heap_buffers) The method Cassandra uses to allocate and manage memtable memory. See Off-heap memtables in Cassandra 2.1. In releases 3.2.0 and 3.2.1, the only option that works is: heap-buffers (On heap NIO (non-blocking I/O) buffers).
- memtable_cleanup_threshold
- (Default: 1/(memtable_flush_writers + 1))note. Ratio used for automatic
                                                memtable flush. Cassandra adds memtable_heap_space_in_mb to memtable_offheap_space_in_mb and
                                                multiplies the total by
                                                  memtable_cleanup_threshold to get
                                                a space amount in MB. When the total amount of
                                                memory used by all non-flushing memtables exceeds
                                                this amount, Cassandra flushes the largest memtable
                                                to disk.For example, consider a node where the total of memtable_heap_space_in_mb and memtable_offheap_space_in_mb is 1000, and memtable_cleanup_threshold is 0.50. The memtable_cleanup amount is500MB. This node has two memtables: Memtable A (150MB) and Memtable B (350MB). When either memtable increases, the total space they use exceeds 500MB and Cassandra flushes the Memtable B to disk.A larger value for memtable_cleanup_threshold means larger flushes, less frequent flushes and potentially less compaction activity, but also less concurrent flush activity, which can make it difficult to keep your disks saturated under heavy write load. This section documents the formula used to calculate the ratio based on the number of memtable_flush_writers. The default value in cassandra.yaml is 0.11, which works if the node has many disks or if you set the node's memtable_flush_writers to8. As another example, if the node uses a single SSD, the value for memttable_cleanup_threshold computes to0.33, based on the minimum memtable_flush_writers value of2.
- file_cache_size_in_mb
- (Default: Smaller of 1/4 heap or 512) Total memory to use for SSTable-reading buffers.
- buffer_pool_use_heap_if_exhausted
- (Default: true)note Indicates whether Cassandra allocates allocate on-heap or off-heap memory when the SSTable buffer pool is exhausted (when the buffer pool has exceeded the maximum memory file_cache_size_in_mb), beyond this amount, Cassandra stops caching buffers, but allocates on request.
- memtable_flush_writers
- (Default: Smaller of number of disks or number of cores with a minimum of 2 and a maximum of 8)note The number of memtable flush writer threads. These threads are blocked by disk I/O, and each one holds a memtable in memory while blocked. If your data directories are backed by SSDs, increase this setting to the number of cores.
Cache and index settings
- column_index_size_in_kb
- (Default: 64) Granularity of the index of rows within a partition. For huge rows, decrease this setting to improve seek time. If you use key cache, be careful not to make this setting too large because key cache will be overwhelmed. If you're unsure of the size of the rows, it's best to use the default setting.
- index_summary_capacity_in_mb
- (Default: 5% of the heap size [empty])note Fixed memory pool size in MB for SSTable index summaries. If the memory usage of all index summaries exceeds this limit, any SSTables with low read rates shrink their index summaries to meet this limit. This is a best-effort process. In extreme conditions, Cassandra may use more than this amount of memory.
- index_summary_resize_interval_in_minutes
- (Default: 60 minutes) How frequently index summaries should be re-sampled. Re-sampling is done periodically to redistribute memory from the fixed-size pool to SSTables proportional their recent read rates. To disable, set to -1. This setting leaves existing index summaries at their current sampling level.
Disks settings
- stream_throughput_outbound_megabits_per_sec
- (Default: 200 Mbps)note Throttle for the throughput of all outbound streaming file transfers on a node. Cassandra does mostly sequential I/O when streaming data during bootstrap or repair. This can saturate the network connection and degrade client (RPC) performance.
- inter_dc_stream_throughput_outbound_megabits_per_sec
- (Default: unset)note Throttle for all streaming file transfers between datacenters, and for network stream traffic as configured with stream_throughput_outbound_megabits_per_sec.
- trickle_fsync
- (Default: false) When set to
                                                  true, causes fsync to force the operating system to flush the dirty buffers at the set interval trickle_fsync_interval_in_kb. Enable this parameter to prevent sudden dirty buffer flushing from impacting read latencies. Recommended for use with SSDs, but not with HDDs.
- windows_timer_interval
- (Default: 1) The default Windows
                                                kernel timer and scheduling resolution is 15.6ms for
                                                power conservation. Lowering this value on Windows
                                                can provide much tighter latency and better
                                                throughput. However, some virtualized environments
                                                may see a negative performance impact from changing
                                                this setting below the system default. The
                                                  sysinternals clockrestool can confirm your system's default setting.
Advanced properties
Properties for advanced users or properties that are less commonly used.
Advanced initialization properties
- auto_bootstrap
- (Default: true) This setting has been
                                                removed from default configuration. It causes new
                                                (non-seed) nodes  migrate the right data to
                                                themselves automatically. When initializing a fresh
                                                cluster without data, add
                                                  auto_bootstrap:
                                                  false.Related information: Initializing a multiple node cluster (single datacenter) and Initializing a multiple node cluster (multiple datacenters). 
- batch_size_warn_threshold_in_kb
- (Default: 5KB per batch) Causes
                                                Cassandra to log a WARN message when any batch size
                                                exceeds this value in kilobytes. CAUTION:Increasing this threshold can lead to node instability.
- batch_size_fail_threshold_in_kb
- (Default: 50KB per batch) Cassandra
                                                fails any batch whose size exceeds this setting. The
                                                default value is 10X the value of
                                                  batch_size_warn_threshold_in_kb.
- unlogged_batch_across_partitions_warn_threshold
- (Default: 10partitions per batch) Causes Cassandra to log a WARN message on any batches not of type LOGGED that span across more partitions than this limit. The default value is 10 partitions.
- cdc_enabled
- (Default: commented out) Enable /
                                                disable CDC functionality on a per-node basis. This
                                                modifies the logic used for write path allocation
                                                rejection (standard: never reject. cdc: reject
                                                Mutation containing a CDC-enabled table if at space
                                                limit in
                                                  cdc_raw_directory).Important: Do not enable CDC on a mixed-version cluster. Upgrade all nodes to Cassandra 3.8 before enabling and restarting the cluster.
- broadcast_address
- (Default: listen_address)note
                                                The "public" IP
                                                  address this node uses to broadcast to other nodes
                                                  outside the network or across regions in
                                                  multiple-region EC2 deployments. If this property
                                                  is commented out, the node uses the same IP
                                                  address or hostname as
                                                  listen_address. A node does not need a separatebroadcast_addressin a single-node or single-datacenter installation, or in an EC2-based network that supports automatic switching between private and public communication. It is necessary to set a separatelisten_addressandbroadcast_addresson a node with multiple physical network interfaces or other topologies where not all nodes have access to other nodes by their private IP addresses. For specific configurations, see the instructions for listen_address.
- listen_on_broadcast_address
- (Default: false) If this node uses
                                                multiple physical network interfaces, set a unique
                                                IP address for broadcast_address and set
                                                  listen_on_broadcast_addressto true. This enables the node to communicate on both interfaces.Set this property to false if the node is on a network that automatically routes between public and private networks, as Amazon EC2 does. For configuration details, see the instructions for listen_address. 
- initial_token
- (Default: disabled) Set this property for single-node-per-token architecture, in which a node owns exactly one contiguous range in the ring space. Setting this property overrides num_tokens.
- num_tokens
- (Default: 256) note
                                                The number of
                                                  tokens randomly assigned to this node in a cluster
                                                  that uses virtual nodes (vnodes). This setting
                                                is evaluated in relation to the
                                                  num_tokensset on other nodes in the cluster. If this node'snum_tokensvalue is higher than the values on other nodes, the vnode logic assigns this node a larger proportion of data relative to other nodes. In general, if all nodes have equal hardware capability, each one should have the samenum_tokensvalue . The recommended value is 256. If this property is commented out (#num_tokens), Cassandra uses 1 (equivalent to#num_tokens : 1) for legacy compatibility and assigns tokens using the initial_token property.
- allocate_tokens_keyspace
- (Default: KEYSPACE) Enables automatic
                                                allocation of num_tokenstokens for this node. The allocation algorithm attempts to choose tokens in a way that optimizes replicated load over the nodes in the datacenter for the replication strategy used by the specified KEYSPACE. The load assigned to each node will near proportional to its number of vnodes.
- partitioner
- (Default:
                                                  org.apache.cassandra.dht.Murmur3Partitioner)
                                                Sets the class that distributes rows (by partition
                                                key) across all nodes in the cluster. Any
                                                  IPartitioner may be used,
                                                including your own as long as it is in the class
                                                path. For new clusters use the default
                                                  partitioner.Cassandra provides the following partitioners for backwards compatibility:- RandomPartitioner
- ByteOrderedPartitioner (deprecated)
- OrderPreservingPartitioner (deprecated)
 Related information: Partitioners 
- tracetype_query_ttl
- (Default: 86400) TTL for different trace types used during logging of the query process
- tracetype_repair_ttl
- (Default: 604800) TTL for different trace types used during logging of the repair process.
Advanced automatic backup setting
- auto_snapshot
- (Default: true) Enables or disables
                                                whether Cassandra takes a snapshot of the data
                                                before truncating a keyspace or dropping a table. To
                                                prevent data loss, Datastax strongly advises using
                                                the default setting. If you set
                                                  auto_snapshottofalse, you lose data on truncation or drop.
Key caches and global row properties
When creating or modifying tables, you can enable or disable the key cache (partition key cache) or row cache for that table by setting the caching parameter. Other row and key cache tuning and configuration options are set at the global (node) level. Cassandra uses these settings to automatically distribute memory for each table on the node based on the overall workload and specific table usage. You can also configure the save periods for these caches globally.
Related information: Configuring caches
- key_cache_keys_to_save
- (Default: disabled - all keys are saved)note Number of keys from the key cache to save.
- key_cache_save_period
- (Default: 14400 seconds [4 hours]) Duration in seconds that keys are kept in cache. Caches are saved to saved_caches_directory. Saved caches greatly improve cold-start speeds and have relatively little effect on I/O.
- key_cache_size_in_mb
- (Default: empty) A global cache setting for the maximum
                                                size of the key cache in memory (for all tables). If
                                                no value is set, the cache is set to the smaller of
                                                5% of the available heap, or 100MB. To disable set
                                                to 0. Related information: setcachecapacity, Enabling and configuring caching. 
- column_index_cache_size_in_kb
- (Default: 2) A threshold for the total
                                                size of all index entries for a partition that
                                                Cassandra stores in the partition key cache. If the
                                                total size of all index entries for a partition
                                                exceeds this amount, Cassandra stops putting entries
                                                for this partition into the partition key cache.
                                                This limitation prevents index entries from large
                                                partitions from taking up all the space in the
                                                partition key cache (which is controlled by
                                                  key_cache_size_in_mb).
- row_cache_class_name
- (Default: disabled  - row cache is not
                                                  enabled)note The classname of the row
                                                cache provider to use. Valid values:
                                                  OHCProvider(fully off-heap) orSerializingCacheProvider(partially off-heap).
- row_cache_keys_to_save
- (Default: disabled - all keys are saved)note Number of keys from the row cache to save.
- row_cache_size_in_mb
- (Default: 0- disabled) Maximum size of the row cache in memory. The row cache can save more time than key_cache_size_in_mb,, but it is space-intensive because it contains the entire row. Use the row cache only for hot rows or static rows. If you reduce the size, you may not get you hottest keys loaded on start up.
- row_cache_save_period
- (Default: 0- disabled) The number of seconds that rows are kept in cache. Caches are saved to saved_caches_directory. This setting has limited use as described in row_cache_size_in_mb.
Counter caches properties
Counter cache helps to reduce counter locks' contention for hot counter cells. In case of RF = 1 a counter cache hit causes Cassandra to skip the read before write entirely. With RF > 1 a counter cache hit still helps to reduce the duration of the lock hold, helping with hot counter cell updates, but does not allow skipping the read entirely. Only the local (clock, count) tuple of a counter cell is kept in memory, not the whole counter, so it is relatively cheap.
- counter_cache_size_in_mb
- (Default value: empty)note When no value is set, Cassandra uses the smaller of minimum of 2.5% of Heap or 50MB. If your system performs counter deletes and relies on low gc_grace_seconds, you should disable the counter cache. To disable, set to 0.
- counter_cache_save_period
- (Default: 7200 seconds [2 hours]) the amount of time after which Cassandra saves the counter cache (keys only). Cassandra saves caches to saved_caches_directory.
- counter_cache_keys_to_save
- (Default value: disabled)note Number of keys from the counter cache to save. When this property is disabled, Cassandra saves all keys.
Tombstone settings
When executing a scan, within or across a partition, Cassandra must keep tombstones in memory to allow them to return to the coordinator. The coordinator uses tombstones to ensure that other replicas know about the deleted rows. Workloads that generate numerous tombstones may cause performance problems and exhaust the server heap. See Cassandra anti-patterns: Queues and queue-like datasets. Adjust these thresholds only if you understand the impact and want to scan more tombstones. You can adjust these thresholds at runtime using the StorageServiceMBean.
Related information: Cassandra anti-patterns: Queues and queue-like datasets
- tombstone_warn_threshold
- (Default: 1000) Cassandra issues a warning if a query scans more than this number of tombstones.
- tombstone_failure_threshold
- (Default: 100000) Cassandra aborts a query if it scans more than this number of tombstones.
Network timeout settings
- range_request_timeout_in_ms
- (Default: 10000 milliseconds) The number of milliseconds that the coordinator waits for sequential or index scans to complete before timing it out.
- read_request_timeout_in_ms
- (Default: 5000 milliseconds) The number of milliseconds that the coordinator waits for read operations to complete before timing it out.
- counter_write_request_timeout_in_ms
- (Default: 5000 milliseconds) The number of milliseconds that the coordinator waits for counter writes to complete before timing it out.
- cas_contention_timeout_in_ms
- (Default: 1000 milliseconds) The number of milliseconds during which the coordinator continues to retry a CAS (compare and set) operation that contends with other proposals for the same row. If the coordinator cannot complete the operation within this timespan, it aborts the operation.
- cqlTruncateequest_timeout_in_ms
- (Default: 60000 milliseconds) The number of milliseconds that the coordinator waits for a truncate (the removal of all data from a table) to complete before timing it out. The long default value allows Cassandra to take a snapshot before removing the data. If auto_snapshot is disabled (not recommended), you can reduce this time.
- write_request_timeout_in_ms
- (Default: 2000 milliseconds) The number
                                                of milliseconds  that the coordinator waits for a
                                                write operations to complete before timing it out.
                                                  Related information: Hinted Handoff: repair during write path 
- request_timeout_in_ms
- (Default: 10000 milliseconds) The
                                                default timeout value for other miscellaneous
                                                  operations.Related information: Hinted Handoff: repair during write path 
Inter-node settings
- cross_node_timeout
- (Default: false) Enables or disables
                                                operation timeout information exchange between nodes
                                                (to accurately measure request timeouts). If this
                                                property is disabled, Cassandra assumes the requests
                                                are forwarded to the replica instantly by the
                                                coordinator, which means that under overload
                                                conditions extra time is required for processing
                                                already-timed-out requests.CAUTION:Before enabling this property make sure NTP (network time protocol) is installed and the times are synchronized among the nodes.
- internode_send_buff_size_in_bytes
- (Default: N/A)note The sending socket buffer
                                                size in bytes for inter-node calls.The buffer size set by this parameter and internode_recv_buff_size_in_bytes is limited bynet.core.wmem_max. If this property is not set,net.ipv4.tcp_wmemdetermines the buffer size. See man tcp and:- /proc/sys/net/core/wmem_max
- /proc/sys/net/core/rmem_max
- /proc/sys/net/ipv4/tcp_wmem
- /proc/sys/net/ipv4/tcp_wmem
 Related information: TCP settings 
- internode_recv_buff_size_in_bytes
- (Default: N/A)noteThe receiving socket buffer size in bytes for inter-node calls.
- internode_compression
- (Default: all) Controls whether traffic
                                                between nodes is compressed. Valid values: - allCompresses all traffic. 
- dc
                                                  Compresses traffic between datacenters only. 
- noneNo compression. 
 
- all
- inter_dc_tcp_nodelay
- (Default: false) Enable this property
                                                or disable tcp_nodelay for
                                                inter-datacenter communication. If this property is
                                                disabled, the network sends larger, but fewer,
                                                network packets. This reduces overhead from the TCP
                                                protocol itself. However, disabling
                                                  inter_dc_tcp_nodelaymay increase latency by blocking cross data-center responses.
- streaming_socket_timeout_in_ms
- (Default: 3600000 - 1 hour)note Enables or disables socket timeout for streaming operations. If a streaming times out by exceeding this number of milliseconds, Cassandra retries it from the start of the current file. Setting this value too low can result in a significant amount of data re-streaming.
Native transport (CQL Binary Protocol)
- start_native_transport
- (Default: true) Enables or disables the native transport server. This server uses the same address as the rpc_address, but the port it uses is different from rpc_port. See native_transport_port.
- native_transport_max_threads
- (Default: 128)note The maximum number of
                                                thread handling requests. Similar to rpc_max_threads, but this property differs
                                                as follows: - The default for
                                                  native_transport_max_threadsis 128; the default forrpc_max_threadsis unlimited.
- There is no corresponding
                                                  native_transport_min_threads.
- Cassandra stops idle native transport threads after 30 seconds.
 
- The default for
                                                  
- native_transport_max_frame_size_in_mb
- (Default: 256MB) The maximum allowed size of a frame. Frame (requests) larger than this are rejected as invalid.
- native_transport_max_concurrent_connections
- (Default: -1) The maximum number of concurrent client connections. The default value of -1 means unlimited.
- native_transport_max_concurrent_connections_per_ip
- (Default: -1) The maximum number of concurrent client connections per source IP address. The default value of -1 means unlimited.
RPC (remote procedure call) settings
Settings for configuring and tuning client connections.
- broadcast_rpc_address
- (Default: unset)note The RPC address for broadcast to drivers and other Cassandra nodes. This cannot be set to 0.0.0.0. If left blank, Cassandra uses the rpc_address or rpc_interface. If rpc_address or rpc_interfaceis set to 0.0.0.0, this property must be set.
- rpc_max_threads
- (Default: unlimited)note Regardless of your choice of RPC server (rpc_server_type), rpc_max_threads dictates the maximum number of concurrent requests in the RPC thread pool. If you are using the parameter sync (see rpc_server_type) it also dictates the number of clients that can be connected. A high number of client connections could cause excessive memory usage for the thread stack. Connection pooling on the client side is highly recommended. Setting a rpc_max_threads acts as a safeguard against misbehaving clients. If the number of threads reaches the maximum, Cassandra blocks additional connections until a client disconnects.
- rpc_min_threads
- (Default: 16)noteThe minimum thread pool size for remote procedure calls.
- rpc_recv_buff_size_in_bytes
- (Default: N/A)note The receiving socket buffer size for remote procedure calls.
- rpc_send_buff_size_in_bytes
- (Default: N/A)note The sending socket buffer size in bytes for remote procedure calls.
- rpc_server_type
- (Default: sync) Cassandra provides
                                                three options for the RPC server. On Windows,
                                                  syncis about 30% slower thanhsha. On Linux,syncandhshaperformance is about the same, buthshauses less memory.- sync: (Default: one thread
                                                  per Thrift connection.)For a very large number of clients, memory is the limiting factor. On a 64-bit JVM, 180KB is the minimum stack size per thread and corresponds to your use of virtual memory. Physical memory may be limited depending on use of stack space. 
- hsha: Half synchronous, half asynchronous. All Thrift clients are handled asynchronously using a small number of threads that does not vary with the number of clients. This mechanism scales well to many clients. The RPC requests are synchronous (one thread per active request).Note: If you select this option, you must change the default value (unlimited) of rpc_max_threads.
- Your own RPC serverYou must provide a fully-qualified class name of an o.a.c.t.TServerFactory that can create a server instance. 
 
- sync: (Default: one thread
                                                  per Thrift connection.)
Advanced fault detection settings
Settings to handle poorly performing or failing components.
- gc_warn_threshold_in_ms
- (Default: 1000) Any GC pause longer
                                                than this interval is logged at the WARN level. (By
                                                default, Cassandra logs any GC pause greater than
                                                200 ms at the INFO level.)Additional information: Configuring logging. 
- dynamic_snitch_badness_threshold
- (Default: 0.1) The performance threshold for dynamically routing client requests away from a poorly performing node. Specifically, it controls how much worse a poorly performing node has to be before the dynamic snitch prefers other replicas over it. A value of 0.2 means Cassandra continues to prefer the static snitch values until the node response time is 20% worse than the best performing node. Until the threshold is reached, incoming requests are statically routed to the closest replica (as determined by the snitch). A value greater than zero for this parameter, with a value of less than 1.0 for read_repair_chance, maximizes cache capacity across the nodes.
- dynamic_snitch_reset_interval_in_ms
- (Default: 600000 milliseconds) Time interval after which Cassandra resets all node scores. This allows a bad node to recover.
- dynamic_snitch_update_interval_in_ms
- (Default: 100 milliseconds) The number of milliseconds between Cassandra's calculation of node scores. Because score calculation is CPU intensive, be careful when reducing this interval.
- hints_flush_period_in_ms
- (Default: 10000) The number of milliseconds Cassandra waits before flushing hints from internal buffers to disk.
- hinted_handoff_enabled
- (Default: true) Enables or disables
                                                hinted handoff. To enable per datacenter, add a
                                                list of datacenters. For example:
                                                  hinted_handoff_enabled: DC1,DC2. A hint indicates that the write needs to be replayed to an unavailable node. Cassandra writes the hint to a hints file on the coordinator node.Related information: Hinted Handoff: repair during write path 
- hinted_handoff_disabled_datacenters
- (Default: none) A blacklist of datacenters that will not perform hinted handoffs. To
                                                disable hinted handoff on a certain datacenter, ,
                                                add its name to this list. For example:
                                                  hinted_handoff_disabled_datacenters: - DC1 - DC2.Related information: Hinted Handoff: repair during write path 
- hinted_handoff_throttle_in_kb
- (Default: 1024) Maximum amount of
                                                traffic per delivery thread in kilobytes per second.
                                                This rate reduces proportionally to the number of
                                                nodes in the cluster. For example, if there are two
                                                nodes in the cluster, each delivery thread  uses the
                                                maximum rate. If there are three, each node
                                                throttles to half of the maximum, since the two
                                                nodes are expected to deliver hints
                                                  simultaneously.Note: When applying this limit, Cassandra computes the hint transmission rate based on the uncompressed hint size, even if internode_compression or hints_compression is enabled.
- max_hint_window_in_ms
- (Default: 10800000 milliseconds [3
                                                hours]) Maximum amount of time during which
                                                Cassandra generates hints for an unresponsive node.
                                                After this interval, Cassandra does not generate any
                                                new hints for the node until it is back up and
                                                responsive. If the node goes down again, Cassandra
                                                starts a new interval. This setting can prevent a
                                                sudden demand for resources when a node is brought
                                                back online and the rest of the cluster attempts to
                                                replay a large volume of hinted writes.Related information: Failure detection and recovery 
- max_hints_delivery_threads
- (Default: 2) Number of threads Cassandra uses to deliver hints. In multiple data-center deployments, consider increasing this number because cross data-center handoff is generally slower.
- hints_compression
- (Default: LZ4Compressor) The compressor for hint files. Supported compressors: LZ, Snappy, and Deflate. If you do not specify a compressor, Cassandra does not compress hints files.
- batchlog_replay_throttle_in_kb
- (Default: 1024KB per second) Total maximum throttle for replaying hints. Throttling is reduced proportionally to the number of nodes in the cluster.
Request scheduler properties
Settings to handle incoming client requests according to a defined policy. If your nodes are overloaded and dropping requests, DataStax recommends that you add more nodes rather than use these properties to prioritize requests.
- request_scheduler
- (Default:
                                                  org.apache.cassandra.scheduler.NoScheduler)
                                                The scheduler to handle incoming client requests
                                                according to a defined policy. This scheduler is
                                                useful for throttling client requests in single
                                                clusters containing multiple keyspaces. This
                                                parameter is specifically for requests from the
                                                client and does not affect inter-node communication.
                                                Valid values: - org.apache.cassandra.scheduler.NoSchedulerCassandra does no scheduling. 
- org.apache.cassandra.scheduler.RoundRobinSchedulerCassandra uses a round robin of client requests to a node with a separate queue for each request_scheduler_id property. 
- Cassandra uses a Java class that implements the RequestScheduler interface.
 
- org.apache.cassandra.scheduler.NoScheduler
- request_scheduler_id
- (Default: keyspace)note The scope of the scheduler's activity. Currently the only valid value is keyspace. See weights.
- request_scheduler_options
- (Default: disabled) A list of
                                                properties that define configuration options for
                                                  request_scheduler: - throttle_limit: The number of in-flight requests per client. Requests beyond this limit are queued up until running requests complete. Recommended value is ((concurrent_reads + concurrent_writes) × 2).
- default_weight: (Default:
                                                  1)note
                                                  How many requests the scheduler handles during each turn of the RoundRobin. 
- weights: (Default:
                                                  Keyspace:
                                                  1) A list of keyspaces. How many requests the scheduler handles during each turn of the RoundRobin, based on the request_scheduler_id. 
 
Thrift interface properties
Legacy API for older clients. CQL is a simpler and better API for Cassandra.
- thrift_framed_transport_size_in_mb
- (Default: 15) Frame size (maximum field length) for Thrift. The frame is the row or part of the row that the application is inserting.
- thrift_max_message_length_in_mb
- (Default: 16) The maximum length of a Thrift message in megabytes, including all fields and internal Thrift overhead (1 byte of overhead for each frame). Calculate message length in conjunction with batches. A frame length greater than or equal to 24 accommodates a batch with four inserts, each of which is 24 bytes. The required message length is greater than or equal to 24+24+24+24+4 (number of frames).
Security properties
Server and client security settings.
- authenticator
- (Default: AllowAllAuthenticator) The
                                                authentication backend. It implements
                                                  IAuthenticator for identifying
                                                users. Available authenticators:- AllowAllAuthenticator:
                                                  Disables authentication; Cassandra performs no checks. 
- PasswordAuthenticatorAuthenticates users with user names and hashed passwords stored in the system_auth.credentials table. Leaving the default replication factor of 1 set for the system_auth keyspace results in denial of access to the cluster if the single replica of the keyspace goes down. For multiple datacenters, be sure to set the replication class to NetworkTopologyStrategy.
 Related information: About Internal authentication 
- AllowAllAuthenticator:
                                                  
- internode_authenticator
- (Default: enabled)note Internode authentication backend. It implements org.apache.cassandra.auth.AllowAllInternodeAuthenticator to allows or disallow connections from peer nodes.
- authorizer
- (Default: AllowAllAuthorizer) The
                                                authorization backend. It implements
                                                  IAuthenticator to limit access
                                                and provide permissions. Available authorizers: - AllowAllAuthorizerDisables authorization: Cassandra allows any action to any user. 
- CassandraAuthorizerStores permissions in system_auth.permissions table. Leaving the default replication factor of 1 set for the system_auth keyspace results in denial of access to the cluster if the single replica of the keyspace goes down. For multiple datacenters, be sure to set the replication class to NetworkTopologyStrategy.
 Related information: Object permissions 
- AllowAllAuthorizer
- role_manager
- (Default: CassandraRoleManager) Part
                                                of the Authentication & Authorization backend
                                                that implements IRoleManager to
                                                maintain grants and memberships between roles. Out
                                                of the box, Cassandra provides
                                                  org.apache.cassandra.auth.CassandraRoleManager,
                                                which stores role information in the system_auth
                                                keyspace. Most functions of the
                                                  IRoleManager require an
                                                authenticated login, so unless the configured
                                                  IAuthenticator actually
                                                implements authentication, most of this
                                                functionality will be unavailable.
                                                  CassandraRoleManagerstores role data in thesystem_authkeyspace. If you use the role manager, increasesystem_authkeyspace replication factor .
- roles_validity_in_ms
- (Default: 2000) Fetching permissions
                                                can be an expensive operation depending on the
                                                authorizer, so this setting allows flexibility.
                                                Validity period for roles cache; set to
                                                  0 to disable. Granted roles are
                                                cached for authenticated sessions in
                                                  AuthenticatedUserand after the period specified here, become eligible for (async) reload. Disabled automatically forAllowAllAuthenticator.
- roles_update_interval_in_ms
- (Default: 2000)  Enable to refresh
                                                interval for roles cache. Defaults to the same value
                                                as roles_validity_in_ms. After this interval, cache entries become eligible for refresh. Upon next access, Cassandra schedules an async reload, and returns the old value until the reload completes. Ifroles_validity_in_msis non-zero, then this must be also.
- credentials_validity_in_ms
- (Default: 2000) How many milliseconds
                                                credentials in the cache remain valid. This cache is
                                                tightly coupled to the provided
                                                PasswordAuthenticator implementation of IAuthenticator. If another
                                                  IAuthenticator implementation
                                                is configured, Cassandra does not use this cache,
                                                and these settings have no effect. Set to
                                                  0 to disable.Related information: Internal authentication, Internal authorization Note: Credentials are cached in encrypted form. This may cause a performance penalty that offsets the reduction in latency gained by caching.
- credentials_update_interval_in_ms
- (Default: same value as
                                                  credentials_validity_in_ms) After
                                                this interval, cache entries become eligible for
                                                refresh. The next time the cache is accessed, the
                                                system schedules an asynchronous reload of the
                                                cache. Until this cache reload is complete, the
                                                cache returns the old values.If credentials_validity_in_ms is nonzero, this property must also be nonzero. 
- permissions_validity_in_ms
- (Default: 2000) How many milliseconds
                                                permissions in cache remain valid. Depending on the
                                                authorizer, such as
                                                  CassandraAuthorizer, fetching
                                                permissions can be resource intensive. This setting
                                                is disabled when set to 0 or when
                                                  AllowAllAuthorizer is set.
                                                  Related information: Object permissions 
- permissions_update_interval_in_ms
- (Default: same value as permissions_validity_in_ms)note If enabled, sets refresh interval for the permissions cache. After this interval, cache entries become eligible for refresh. On next access, Cassandra schedules an async reload and returns the old value until the reload completes. If permissions_validity_in_ms is nonzero, roles_update_interval_in_ms must also be non-zero.
- server_encryption_options
- Enables or disables inter-node encryption. If you enable
                                                  server_encryption_options,
                                                you must also generate keys and provide the
                                                appropriate key and truststore locations and
                                                passwords. There are no custom encryption options
                                                currently enabled for Cassandra. Available
                                                  options:- internode_encryption:
                                                  (Default: none) Enables or
                                                  disables encryption of inter-node communication
                                                  using the
                                                  TLS_RSA_WITH_AES_128_CBC_SHAcipher suite for authentication, key exchange, and encryption of data transfers. Use the DHE/ECDHE ciphers, such asTLS_DHE_RSA_WITH_AES_128_CBC_SHAif running in (Federal Information Processing Standard) FIPS 140 compliant mode. Available inter-node options:- allEncrypt all inter-node communications. 
- 
                                                  noneNo encryption. 
- dcEncrypt the traffic between the datacenters (server only). 
- rackEncrypt the traffic between the racks (server only). 
 
- all
- keystore: (Default:
                                                  conf/.keystore) The location of a Java keystore (JKS) suitable for use with Java Secure Socket Extension (JSSE), which is the Java version of the Secure Sockets Layer (SSL), and Transport Layer Security (TLS) protocols. The keystore contains the private key used to encrypt outgoing messages. 
- keystore_password:
                                                  (Default: cassandra)
                                                  Password for the keystore. 
- truststore: (Default:
                                                  conf/.truststore) Location of the truststore containing the trusted certificate for authenticating remote servers. 
- truststore_password:
                                                  (Default: cassandra)
                                                  Password for the truststore. 
 The passwords used in these options must match the passwords used when generating the keystore and truststore. For instructions on generating these files, see Creating a Keystore to Use with JSSE. The advanced settings: - protocol: (Default: TLS)
- algorithm: (Default: SunX509)
- store_type: (Default: JKS)
- cipher_suites: (Default: TLS_RSA_WITH_AES_128_CBC_SHA,TLS_RSA_WITH_AES_256_CBC_SHA)
- require_client_auth:
                                                  (Default: false) Enables or disables certificate authentication. 
- require_endpoint_verification:
                                                  (Default: false)Enables or disables host name verification. 
 Related information: Node-to-node encryption 
- internode_encryption:
                                                  (Default: none) Enables or
                                                  disables encryption of inter-node communication
                                                  using the
                                                  
- client_encryption_options
- Enables or disables client-to-node encryption. You must
                                                also generate keys and provide the appropriate key
                                                and truststore locations and passwords. There are no
                                                custom encryption options are currently enabled for
                                                Cassandra. Available options:- enabled: (Default:
                                                  false) To enable, set to true. 
- keystore: (Default:
                                                  conf/.keystore) The location of a Java keystore (JKS) suitable for use with Java Secure Socket Extension (JSSE), which is the Java version of the Secure Sockets Layer (SSL), and Transport Layer Security (TLS) protocols. The keystore contains the private key used to encrypt outgoing messages. 
- keystore_password:
                                                  (Default: cassandra)
                                                  Password for the keystore. This must match the password used when generating the keystore and truststore. 
- require_client_auth:
                                                  (Default: false) Enables or disables certificate authentication. (Available starting with Cassandra 1.2.3.) 
- truststore: (Default:
                                                  conf/.truststore) Set this property if require_client_auth is true.
- truststore_password:
                                                  truststore_password
                                                  Set if require_client_auth is true.
 Advanced settings: - protocol: (Default: TLS)
- algorithm: (Default: SunX509)
- store_type: (Default: JKS)
- cipher_suites: (Default: TLS_RSA_WITH_AES_128_CBC_SHA,TLS_RSA_WITH_AES_256_CBC_SHA)
 Related information: Client-to-node encryption 
- enabled: (Default:
                                                  false) 
- transparent_data_encryption_options
- Enables encryption of data at rest (on-disk).
                                                Recommendation: download and install the Java Cryptography
                                                  Extension (JCE) Unlimited Strength Jurisdiction
                                                  Policy Files for your version of the
                                                  JDK.
Available options: - enabled: (Default: false)
- chunk_length_kb: (Default: 64)
- cipher: options:- AES
- CBC
- PKCS5Padding
 
- key_alias: testing:1
- 
                                                  iv_length:
                                                  16Note: iv_length is commented out in the default cassandra.yaml file. Uncomment only if cipher is set to AES. The value must be 16 (bytes).
- key_provider: - class_name:
                                                  org.apache.cassandra.security.JKSKeyProviderparameters: - keystore: conf/.keystore
- keystore_password: cassandra
- store_type: JCEKS
- key_password: cassandra
 
 
- class_name:
                                                  org.apache.cassandra.security.JKSKeyProvider
 
- ssl_storage_port
- (Default: 7001) The SSL port for encrypted communication. Unused unless enabled in encryption_options.
- native_transport_port_ssl
- (Default: 9142) In Cassandra 3.0 and
                                                later, an additional dedicated port can be
                                                designated for encryption. If client encryption is
                                                enabled and
                                                  native_transport_port_sslis disabled, thenative_transport_port(default: 9042) will encrypt all traffic. To use both unencrypted and encrypted traffic, enablenative_transport_port_ssl
| Package installations | /etc/cassandra/cassandra-env.sh | 
| Tarball installations | install_location/conf/cassandra-env.sh | 

