Monitor SAI indexes

Use these tools to monitor secondary indexes that were created with Storage-Attached Indexing (SAI), including metrics and options for pre-configured dashboards.

CQL virtual tables

System virtual tables provide information about the status of SAI indexes.

The system_views.indexes provides information at the column index level. This table includes the index name, number of indexed SSTables, disk usage, and the index state. From the index state, the data reveals if the index is currently building, and whether the index can be queried.

When you DROP or recreate an SAI index, you are not blocked from entering queries that do not use the index. However, you cannot use that SAI index (based on the same column) until it has finished building and is queryable. To determine the current state of a given index, query the system_views.indexes virtual table. Example:

SELECT is_queryable,is_building FROM system_views.indexes
  WHERE keyspace_name='<keyspace>'
  AND table_name='<table>'
  AND index_name='<index>';

The system_views.sstable_indexes describes individual SSTable indexes. This table includes information about disk size, minimum and maximum row ID, the minimum and maximum ring token, and the write-time version of the index.

The system_views.sstable_index_segments describes the segments of the SSTable indexes. This table exposes the segment row ID offset and most of the information in the SSTable-level virtual table, specifically at a segment granularity.

For more details, refer to Virtual tables.

DataStax Enterprise tracing

SAI provides tracing capability just like other DSE database components. Information is captured by the system_traces keyspace. You can enable tracing in CQLSH with TRACING ON, or in the DataStax driver with statement.enableTracing().

The number of rows filtered by a particular query will show up in the CQL query trace. Example:

Index query accessed memtable indexes, 2 SSTable indexes, and 2 segments, post-filtered 14 rows in 14 partitions, and took 88582 microseconds.

For details about DSE tracing, refer to CQL TRACING.

SAI metrics

SAI provides a number of metrics to help you monitor the health of your indexes.

The metrics are categorized as:

  • Global indexing metrics

  • Table query metrics

  • Per query metrics

  • Key fetch metrics

  • Offset fetch metrics

  • Token fetch metrics

  • Column query metrics per index

  • Terms metrics per index

  • Range slice metrics

For example, use metrics to get the current count of total partition reads since the node started for cycling.cyclist_semi_pro. The keyspace and table were defined in SAI quickstart. This metric’s ObjectName: org.apache.cassandra.metrics:type=StorageAttachedIndex,keyspace=cycling,table=cyclist_semi_pro,scope=TableQueryMetrics,name=TotalPartitionReads.

The metrics are exposed via JMX, so any JMX-based tool can be used to monitor. Common tools include:

Global indexing metrics

ObjectName: org.apache.cassandra.metrics,type=StorageAttachedIndex,name=<metric>

The global indexing metrics for this node are:

ColumnIndexBuildsInProgress

The number of individual on-disk column indexes currently being built.

SegmentBufferSpaceLimitBytes

The limit on heap used to buffer SSTable index segments during compaction and index builds.

SegmentBufferSpaceUsedBytes

The heap currently being used to buffer SSTable index segments during compaction and index builds.

At any given time, the minimum size for a flushing segment, in bytes, is (SegmentBufferSpaceLimitBytes / ColumnIndexBuildsInProgress).

Index group metrics

ObjectName: org.apache.cassandra.metrics:type=StorageAttachedIndex,keyspace=<keyspace>,table=<table>,scope=IndexGroupMetrics,name=<metric>

The index group metrics for the given keyspace and table:

DiskUsedBytes

Size in bytes on disk for the given table’s SAI indices.

IndexFileCacheBytes

Size in bytes of memory used by the on-disk data structure of the per-column indices.

OpenIndexFiles

Number of open index files for the given table’s SAI indices.

Key fetch metrics

ObjectName: org.apache.cassandra.metrics:type=StorageAttachedIndex,keyspace=<keyspace>,table=<table>,scope=KeyFetch,name=<metric>

The key fetch metrics for the given keyspace and table:

ChunkCacheHitRate

All-time chunk cache hit rate for keys during queries against the given table.

TotalChunkCacheLookups

All-time chunk cache lookups for keys during queries against the given table.

TotalChunkCacheMisses

All-time chunk cache misses for keys during queries against the given table.

`ChunkCache(One

Five

Fifteen)HitRate`

<N>-minute chunk cache hit rate for keys during queries against the given table.

Offset fetch metrics

ObjectName: org.apache.cassandra.metrics:type=StorageAttachedIndex,keyspace=<keyspace>,table=<table>,scope=OffsetFetch,name=<metric>

The offset fetch metrics for the given keyspace and table:

ChunkCacheHitRate

All-time chunk cache hit rate for partition key SSTable offset fetches during queries against the given table.

TotalChunkCacheLookups

All-time chunk cache lookups for partition key SSTable offset fetches during queries against the given table.

TotalChunkCacheMisses

All-time chunk cache misses for partition key SSTable offset fetches during queries against the given table.

`ChunkCache(One

Five

Fifteen)HitRate`

<N>-minute chunk cache hit rate for partition key SSTable offset fetches during queries against the given table.

Per query metrics

ObjectName: org.apache.cassandra.metrics:type=StorageAttachedIndex,keyspace=<keyspace>,table=<table>,scope=PerQuery,name=<metric>

The per query metrics for the given keyspace and table include:

RowsFiltered

A histogram of the number of rows post-filtered per query since the node started.

QueryLatency

Overall query latency percentiles (in microseconds) and one/five/fifteen minute query throughput.

PartitionReads

Histogram over the number of partitions read per query.

SSTableIndexesHit

Histogram over the number of SSTable indexes read per query.

KDTreeChunkCacheLookups

Histogram over the number of chunk cache lookups while reading kd-tree index files per query.

KDTreeChunkCacheMisses

Histogram over the number of chunk cache misses while reading kd-tree index files per query.

Table query metrics

ObjectName: org.apache.cassandra.metrics:type=StorageAttachedIndex,keyspace=<keyspace>,table=<table>,scope=TableQueryMetrics,name=<metric>

The table query metrics for the given keyspace and table:

TotalPartitionReads

Total partition reads by all queries since the node started.

TotalQueriesCompleted

Total number of successfully completed queries since the node started.

TotalQueryTimeouts

Total number of timeouts from queries since the node started.

TotalRowsFiltered

Total number of rows post-filtered by all queries since the node started.

Table state metrics

ObjectName: org.apache.cassandra.metrics:type=StorageAttachedIndex,keyspace=<keyspace>,table=<table>,scope=TableStateMetrics,name=<metric>

The table state metrics for the given keyspace and table:

DiskPercentageOfBaseTable

SAI size on Disk as a percentage of table size per table.

DiskUsedBytes

Size on-disk in bytes of SAI indices per table.

TotalIndexBuildsInProgress

Status of SAI indices per table currently in the is_building state.

TotalIndexCount

Total number of SAI indices per table.

TotalQueryableIndexCount

Status of SAI indices per table currently in the is_querable state.

Token fetch metrics

ObjectName: org.apache.cassandra.metrics:type=StorageAttachedIndex,keyspace=<keyspace>,table=<table>,scope=TokenFetch,name=<metric>

The token fetch metrics for the given keyspace and table:

ChunkCacheHitRate

All-time chunk cache hit rate for partition key token fetches during queries against the given table.

TotalChunkCacheLookups

All-time chunk cache lookups for partition key token fetches during queries against the given table.

TotalChunkCacheMisses

All-time chunk cache misses for partition key token fetches during queries against the given table.

`ChunkCache(One

Five

Fifteen)HitRate`

<N>-minute chunk cache hit rate for partition key token fetches during queries against the given table.

Token skipping metrics

ObjectName: org.apache.cassandra.metrics:type=StorageAttachedIndex,keyspace=<keyspace>,table=<table>,scope=TokenSkipping,name=<metric>

The token skipping metrics for the given keyspace and table:

CacheHits

Number of cache hits from token skipping in a multi-index AND query.

Lookups

Number of lookups from token skipping a multi-index AND query.

Column query metrics for each numeric index

ObjectName: org.apache.cassandra.metrics:type=StorageAttachedIndex,keyspace=<keyspace>,table=<table>,index=<index>,scope=ColumnQueryMetrics,name=<metric>

The column query metrics for the given keyspace, table, and index include:

KDTreeNiceTryLatency

For numeric indexes, such as age_sai_idx in the quickstart examples, this metric may be used to present a histogram of the times spent waiting for chunk cache misses during kd-tree intersection (in microseconds) and one/five/fifteen minute chunk miss throughputs. + [NOTE] ==== The throughputs are zero if there are no cache misses during kd-tree intersection. ====

Column query metrics for each string index

ObjectName: org.apache.cassandra.metrics:type=StorageAttachedIndex,keyspace=<keyspace>,table=<table>,index=<index>,scope=ColumnQueryMetrics,name=<metric>

The column query metrics for the given keyspace, table, and index include:

TermsLookupLatency

For string indexes, such as country_sai_idx in the quickstart examples, this metric shows terms lookup latency percentiles (in microseconds) per one/five/fifteen minute query throughput.

Terms metrics for each string index

ObjectName: org.apache.cassandra.metrics:type=StorageAttachedIndex,keyspace=<keyspace>,table=<table>,index=<index>,scope=Terms,name=<metric>

For string indexes, the terms metrics for the given keyspace, table, and index:

ChunkCacheHitRate

All-time chunk cache hit rate for terms during string index queries that used the given index.

TotalChunkCacheLookups

All-time chunk cache lookups for terms during string index queries that used the given index.

TotalChunkCacheMisses

All-time chunk cache misses for terms during string index queries that used the given index.

`ChunkCache(One

Five

Fifteen)HitRate`

<N>-minute chunk cache hit rate for terms during string index queries that used the given index.

Range slice metrics

ObjectName: org.apache.cassandra.metrics:type=ClientRequest,scope=RangeSlice,name=<metric>

The RoundTripsPerReadHistogram metric tracks the number of round-trip requests sent for range query commands from the coordinator. Fewer requests typically mean the server is operating more efficiently than ones requiring more requests to satisfy the same range queries.

Latency metric tracks the min, max, mean as well as a set of percentiles for range read requests latency. Timeouts metric tracks the number of timeouts for range read requests.

Was this helpful?

Give Feedback

How can we improve the documentation?

© 2024 DataStax | Privacy policy | Terms of use

Apache, Apache Cassandra, Cassandra, Apache Tomcat, Tomcat, Apache Lucene, Apache Solr, Apache Hadoop, Hadoop, Apache Pulsar, Pulsar, Apache Spark, Spark, Apache TinkerPop, TinkerPop, Apache Kafka and Kafka are either registered trademarks or trademarks of the Apache Software Foundation or its subsidiaries in Canada, the United States and/or other countries. Kubernetes is the registered trademark of the Linux Foundation.

General Inquiries: +1 (650) 389-6000, info@datastax.com