Skip to content

Latest commit

 

History

History
1444 lines (1296 loc) · 103 KB

stats.org

File metadata and controls

1444 lines (1296 loc) · 103 KB

EP Stats

1 Getting Started

For introductory information on stats within Couchbase, start with the Couchbase server documentations.

2 Stats Definitions

2.1 Toplevel Stats

StatDescription
uuidThe unique identifier for the bucket
ep_versionVersion number of ep_engine
ep_startup_timeSystem-generated engine startup time
ep_data_ageSeconds since most recently
stored object was modified
ep_data_age_highwatep_data_age high water mark
ep_data_write_failedTotal compaction and commit failures
ep_num_workersGlobal number of shared worker threads
ep_bucket_priorityPriority assigned to the bucket
ep_total_deduplicatedTotal number of items de-duplicated
when queued to CheckpointManager
ep_total_deduplicated_flusherTotal number of items de-duplicated
when processed at Flusher
ep_total_enqueuedTotal number of items queued for
persistence
ep_total_new_itemsTotal number of persisted new items
ep_total_del_itemsTotal number of persisted deletions
ep_total_persistedTotal number of items persisted
ep_item_flush_failedNumber of times an item failed to
flush due to storage errors
ep_item_commit_failedNumber of times a transaction failed
to commit due to storage errors
ep_item_begin_failedNumber of times a transaction failed
to start due to storage errors
ep_expired_accessNumber of times an item was expired on
application access.
ep_expired_compactorNumber of times an item was expired by
the ep engine compactor
ep_expired_pagerNumber of times an item was expired by
ep engine item pager
ep_item_flush_expiredNumber of times an item is not flushed
due to the expiry of the item
ep_queue_sizeNumber of items queued for storage
ep_flusher_todoNumber of items currently being
written
ep_flusher_stateCurrent state of the flusher thread
ep_commit_numTotal number of write commits
ep_commit_timeNumber of milliseconds of most recent
commit
ep_commit_time_totalCumulative milliseconds spent
committing
ep_vbucket_delNumber of vbucket deletion events
ep_vbucket_del_failNumber of failed vbucket deletion
events
ep_vbucket_del_max_walltimeMax wall time (µs) spent by deleting
a vbucket
ep_vbucket_del_avg_walltimeAvg wall time (µs) spent by deleting
a vbucket
ep_pending_compactionsFor persistent buckets this is the count
of compaction tasks.
ep_rollback_countNumber of rollbacks on consumer
ep_flush_duration_totalCumulative milliseconds spent flushing
ep_num_ops_get_metaNumber of getMeta operations
ep_num_ops_set_metaNumber of setWithMeta operations
ep_num_ops_del_metaNumber of delWithMeta operations
ep_num_ops_set_meta_res_failedNumber of setWithMeta ops that failed
conflict resolution
ep_num_ops_del_meta_res_failedNumber of delWithMeta ops that failed
conflict resolution
ep_num_ops_set_ret_metaNumber of setRetMeta operations
ep_num_ops_del_ret_metaNumber of delRetMeta operations
ep_num_ops_get_meta_on_set_metaNum of background getMeta operations
spawn due to setWithMeta operations
curr_itemsNum items in active vbuckets (temp +
live)
curr_temp_itemsNum temp items in active vbuckets
curr_items_totNum current items including those not
active (replica, dead and pending
states)
ep_kv_sizeMemory used to store item metadata,
keys and values, no matter the
vbucket’s state. If an item’s value is
ejected, this stats will be
decremented by the size of the item’s
value.
ep_blob_numThe number of blob objects in the cache
ep_blob_overheadThe “unused” memory caused by the
allocator returning bigger chunks than
requested
ep_value_sizeMemory used to store values for
resident keys
ep_storedval_sizeMemory used by storedval objects
ep_storedval_overheadThe “unused” memory caused by the
allocator returning bigger chunks than
requested
ep_storedval_numThe number of storedval objects
allocated
ep_overheadExtra memory used by transient data
like persistence queues, replication
queues, checkpoints, etc
ep_item_numThe number of item objects allocated
ep_mem_low_watLow water mark for auto-evictions
ep_mem_low_wat_percentLow water mark (as a percentage)
ep_mem_high_watHigh water mark for auto-evictions
ep_mem_high_wat_percentHigh water mark (as a percentage)
ep_total_cache_sizeThe total byte size of all items, no
matter the vbucket’s state, no matter
if an item’s value is ejected
ep_oom_errorsNumber of times unrecoverable OOMs
happened while processing operations
ep_tmp_oom_errorsNumber of times temporary OOMs
happened while processing operations
ep_mem_tracker_enabledTrue if memory usage tracker is
enabled
ep_bg_fetchedNumber of items fetched from disk
ep_bg_fetch_avg_read_amplificationAverage read amplification for all
background fetch operations - ratio of
read()s to documents fetched.
ep_bg_meta_fetchedNumber of meta items fetched from disk
ep_bg_remaining_itemsNumber of remaining bg fetch items
ep_bg_remaining_jobsNumber of remaining bg fetch jobs
ep_num_pager_runsNumber of times we ran pager loops
to seek additional memory
ep_num_expiry_pager_runsNumber of times we ran expiry pager
loops to purge expired items from
memory/disk
ep_num_freq_decayer_runsNumber of times we ran the freq decayer
task because a frequency counter has
become saturated
ep_num_access_scanner_runsNumber of times we ran accesss scanner
to snapshot working set
ep_num_access_scanner_skipsNumber of times accesss scanner task
decided not to generate access log
ep_access_scanner_num_itemsNumber of items that last access
scanner task swept to access log.
ep_access_scanner_task_timeTime of the next access scanner task
(GMT), NOT_SCHEDULED if access scanner
has been disabled
ep_access_scanner_last_runtimeNumber of seconds that last access
scanner task took to complete.
ep_expiry_pager_task_timeTime of the next expiry pager task
(GMT), NOT_SCHEDULED if expiry pager
has been disabled
ep_items_expelled_from_checkpointsNumber of items expelled from
checkpoints. Expelled refers to items
that have been ejected from memory
but are still considered to be part of
the checkpoint.
ep_items_rm_from_checkpointsNumber of items removed from closed
unreferenced checkpoints
ep_num_value_ejectsNumber of times item values got
ejected from memory to disk
ep_num_eject_failuresNumber of items that could not be
ejected
ep_num_not_my_vbucketsNumber of times Not My VBucket
exception happened during runtime
ep_dbnameDB path
ep_pending_opsNumber of ops awaiting pending
vbuckets
ep_pending_ops_totalTotal blocked pending ops since reset
ep_pending_ops_maxMax ops seen awaiting 1 pending
vbucket
ep_pending_ops_max_durationMax time (µs) used waiting on pending
vbuckets
ep_bg_num_samplesThe number of samples included in the
average
ep_bg_min_waitThe shortest time (µs) in the wait
queue
ep_bg_max_waitThe longest time (µs) in the wait
queue
ep_bg_wait_avgThe average wait time (µs) for an item
before it’s serviced by the dispatcher
ep_bg_min_loadThe shortest load time (µs)
ep_bg_max_loadThe longest load time (µs)
ep_bg_load_avgThe average time (µs) for an item to
be loaded from the persistence layer
ep_num_non_residentThe number of non-resident items
ep_bg_waitThe total elapse time for the wait
queue
ep_bg_loadThe total elapse time for items to be
loaded from the persistence layer
ep_allow_data_loss_during_shutdownWhether data loss is allowed during
server shutdown
ep_alog_block_sizeAccess log block size
ep_alog_pathPath to the access log
ep_access_scanner_enabledStatus of access scanner task
ep_alog_sleep_timeInterval between access scanner runs
in minutes
ep_alog_task_timeHour in GMT time when access scanner
task is scheduled to run
ep_backendThe backend that is being used for
data persistence
ep_backfill_mem_thresholdThe maximum percentage of memory that
the backfill task can consume before
it is made to back off.
ep_bfilter_enabledBloom filter use: enabled or disabled
ep_bfilter_key_countMinimum key count that bloom filter
will accomodate
ep_bfilter_fp_probBloom filter’s allowed false positive
probability
ep_bfilter_residency_thresholdResident ratio threshold for full
eviction policy, after which bloom
switches modes from accounting just
non resident items and deletes to
accounting all items
ep_bucket_typeThe bucket type
ep_chk_persistence_remainsNumber of remaining vbuckets for
checkpoint persistence
ep_chk_remover_stimeThe time interval for purging closed
checkpoints from memory
ep_couch_bucketThe name of this bucket
ep_couch_hostThe hostname that the couchdb views
server is listening on
ep_couch_portThe port the couchdb views server is
listening on
ep_couch_reconnect_sleeptimeThe amount of time to wait before
reconnecting to couchdb
ep_data_traffic_enabledWhether or not data traffic is enabled
for this bucket
ep_db_data_sizeTotal size of valid data in db files
ep_db_file_sizeTotal size of the db files
ep_db_prepare_sizeTotal size of SyncWrite prepares in db files
ep_degraded_modeTrue if the engine is either warming
up or data traffic is disabled
ep_exp_pager_enabledTrue if the expiry pager is enabled
ep_exp_pager_stimeThe time interval for purging expired
items from memory
ep_exp_pager_initial_run_timeAn initial start time for the expiry
pager task in GMT
ep_fsync_after_every_n_bytes_writtenIf non-zero, perform an fsync after
every N bytes written to disk
ep_getl_default_timeoutThe default getl lock duration
ep_getl_max_timeoutThe maximum getl lock duration
ep_ht_locksThe amount of locks per vb hashtable
ep_ht_sizeThe initial size of each vb hashtable
ep_max_checkpointsThe expected max number of checkpoints
in each VBucket on a balanced system.
Note: That is not a hard limit on the
single vbucket. That is used (together
with checkpoint_memory_ratio) for
computing checkpoint_max_size, which
triggers checkpoint creation.
ep_max_item_sizeThe maximum value size
ep_max_sizeThe maximum amount of memory this
bucket can use
ep_max_vbucketsThe maximum amount of vbuckets that
can exist in this bucket
ep_mutation_mem_ratioThe ratio of total memory available
that we should start sending temp oom
or oom message when hitting
ep_seqno_persistence_timeoutTimeout for SeqnoPersistence operations
ep_uncommitted_itemsThe amount of items that have not been
written to disk
ep_warmupShows if warmup is enabled / disabled
ep_warmup_batch_sizeThe size of each batch loaded during
warmup
ep_warmup_dupsNumber of Duplicate items encountered
during warmup
ep_warmup_min_items_thresholdPercentage of total items warmed up
before we enable traffic
ep_warmup_min_memory_thresholdPercentage of max mem warmed up before
we enable traffic
ep_warmup_oomThe amount of oom errors that occured
during warmup
ep_warmup_threadThe status of the warmup thread
ep_warmup_timeThe amount of time warmup took
ep_workload_patternWorkload pattern (mixed, read_heavy,
write_heavy) monitored at runtime
ep_defragmenter_intervalHow often defragmenter task should be
run (in seconds).
ep_defragmenter_num_movedNumber of items moved by the
defragmentater task.
ep_defragmenter_num_visitedNumber of items visited (considered
for defragmentation) by the
defragmenter task.
ep_defragmenter_sv_num_movedNumber of StoredValues moved by the
defragmentater task.
ep_item_compressor_intervalHow often item compressor task should
be run (in milliseconds).
ep_item_compressor_num_compressedNumber of items compressed by the
item compressor task.
ep_item_compressor_num_visitedNumber of items visited (considered
for compression) by the
item compressor task.
ep_cursor_dropping_lower_thresholdMemory threshold below which checkpoint
remover will discontinue cursor
dropping.
ep_cursor_dropping_upper_thresholdMemory threshold above which checkpoint
remover will start cursor dropping
ep_cursors_droppedNumber of cursors dropped by the
checkpoint remover
ep_mem_freed_by_checkpoint_removalAmount of memory freed through ckpt
removal
ep_active_hlc_driftThe total absolute drift for all active
vbuckets. This is microsecond
granularity.
ep_active_hlc_drift_countThe number of updates applied to
ep_active_hlc_drift.
ep_replica_hlc_driftThe total absolute drift for all
replica vbuckets. This is microsecond
granularity.
ep_replica_hlc_drift_countThe number of updates applied to
ep_replica_hlc_drift.
ep_active_ahead_exceptionsThe total number of ahead exceptions
for all active vbuckets.
ep_active_behind_exceptionsThe total number of behind exceptions
for all active vbuckets.
ep_replica_ahead_exceptionsThe total number of ahead exceptions
for all replica vbuckets.
ep_replica_behind_exceptionsThe total number of behind exceptions
for all replica vbuckets.
ep_clock_cas_drift_threshold_exceededep_active_ahead_exceptions +
ep_replica_ahead_exceptions
ep_dcp_noop_mandatory_for_v5_featuresIf True,NOOP will be required for using
features like xattrs/collections
ep_retain_erroneous_tombstonesIf True, compactor will retain erroneous
tombstones.
ep_pitr_enabledIf True Point in Time Recovery is
enabled
ep_pitr_max_history_ageThe number of seconds of the oldest
entry to keep as part of compaction
ep_pitr_granularityThe granularity (in seconds) for the
point in time recovery.

2.2 Aggregated KVStore stats. Note the following stats are reported per-shard in ‘kvstore’ stats.

StatDescription
ep_data_read_failedTotal number of get failures
ep_io_total_read_bytesTotal number of bytes read
ep_io_total_write_bytesTotal number of bytes written
ep_io_compaction_read_bytesTotal number of bytes read during compaction
ep_io_compaction_write_bytesTotal number of bytes written during compaction
io_flusher_write_amplificationNumber of bytes written to disk during front-end flushing, divided by the document bytes for each document saved (key + metadata + value).
io_total_write_amplificationNumber of bytes written to disk during front-end flushing and compaction, divided by the document bytes for each document saved (key + metadata + value).

2.3 vBucket total stats

StatDescription
ep_vb_totalTotal vBuckets (count)
curr_items_totTotal number of items
curr_itemsNumber of active items in memory
curr_temp_itemsNumber of temporary items in memory
vb_dead_numNumber of dead vBuckets
ep_diskqueue_itemsTotal items in disk queue
ep_diskqueue_memoryTotal memory used in disk queue
ep_diskqueue_fillTotal enqueued items on disk queue
ep_diskqueue_drainTotal drained items on disk queue
ep_diskqueue_pendingTotal bytes of pending writes
ep_persist_vbstate_totalTotal VB persist state to disk
ep_meta_data_memoryTotal memory used by meta data
ep_meta_data_diskTotal disk used by meta data
ep_checkpoint_memoryMemory of items in all checkpoints
ep_checkpoint_memory_queueMemory of all queued items in all checkpoints
ep_checkpoint_memory_overhead_allocatorMem of all checkpoints struct - from allocator
ep_checkpoint_memory_overhead_allocator_queueMem of all checkpoints queues - from allocator
ep_checkpoint_memory_overhead_allocator_indexMem of all checkpoints index - from allocator
ep_checkpoint_memory_overheadMem of all checkpoints struct
ep_checkpoint_memory_overhead_queueMem of all queues internal struct
ep_checkpoint_memory_overhead_indexMem of all indexes (keys alloc included)
ep_checkpoint_memory_pending_destructionMemory of checkpoint structures awaiting
destruction by a background task
ep_checkpoint_memory_quotaMax allocation allowed in all checkpoints
ep_checkpoint_memory_upper_mark_bytesCheckpoint mem usage that triggers mem recovery
ep_checkpoint_memory_lower_mark_bytesCkpts recovery target, recovery yields when hit

2.3.1 Active vBucket class stats

StatDescription
vb_active_numNumber of active vBuckets
vb_active_curr_itemsNumber of active non-deleted items
vb_active_num_non_residentNumber of non-resident items
vb_active_perc_mem_resident% memory resident
vb_active_ejectNumber of times item values got ejected
vb_active_expiredNumber of times an item was expired
vb_active_ht_memoryMemory overhead of the hashtable
vb_active_ht_memory_overheadMemory overhead of the hashtable
vb_active_itm_memoryTotal memory of all items in active
vBuckets (StoredValue + key + value Blob)
vb_active_meta_data_memoryMetadata memory of all items in active
vBuckets (StoredValue + key)
vb_active_meta_data_diskTotal metadata disk
vb_active_checkpoint_memoryMemory of active items in all checkpoints
vb_active_checkpoint_memory_overheadMemory of all active checkpoints structures
vb_active_ops_createNumber of create operations
vb_active_ops_updateNumber of update operations
vb_active_ops_deleteNumber of delete operations
vb_active_ops_rejectNumber of rejected operations
vb_active_queue_sizeActive items in disk queue
vb_active_backfill_queue_sizeItems in active vbucket backfill queue
vb_active_queue_memoryMemory used for disk queue
vb_active_queue_ageSum of disk queue item age in milliseconds
vb_active_queue_pendingTotal bytes of pending writes
vb_active_queue_fillTotal enqueued items
vb_active_queue_drainTotal drained items
vb_active_rollback_item_countNum of items rolled back
vb_active_sync_write_accepted_countNumber of SyncWrites accepted
vb_active_sync_write_committed_countNumber of SyncWrites committed
vb_active_sync_write_aborted_countNumber of SyncWrites aborted
vb_active_hp_vb_req_sizeNum of async high priority requests

2.3.2 Replica vBucket stats

StatDescription
vb_replica_numNumber of replica vBuckets
vb_replica_curr_itemsNumber of replica non-deleted items
vb_replica_num_non_residentNumber of non-resident items
vb_replica_perc_mem_resident% memory resident
vb_replica_ejectNumber of times item values got ejected
vb_replica_expiredNumber of times an item was expired
vb_replica_ht_memoryMemory overhead of the hashtable
vb_replica_ht_memory_overheadMemory overhead of the hashtable
vb_replica_itm_memoryTotal memory of all items in replica
vBuckets (StoredValue + key + value Blob)
vb_replica_meta_data_memoryMetadata memory of all items in replica
vBuckets (StoredValue + key)
vb_replica_meta_data_diskTotal metadata disk
vb_replica_checkpoint_memoryMemory of replica items in all checkpoints
vb_replica_checkpoint_memory_overheadMemory of all replica checkpoints structures
vb_replica_ops_createNumber of create operations
vb_replica_ops_updateNumber of update operations
vb_replica_ops_deleteNumber of delete operations
vb_replica_ops_rejectNumber of rejected operations
vb_replica_queue_sizeReplica items in disk queue
vb_replica_backfill_queue_sizeItems in replica vbucket backfill queue
vb_replica_queue_memoryMemory used for disk queue
vb_replica_queue_ageSum of disk queue item age in milliseconds
vb_replica_queue_pendingTotal bytes of pending writes
vb_replica_queue_fillTotal enqueued items
vb_replica_queue_drainTotal drained items
vb_replica_rollback_item_countNum of items rolled back
vb_replica_sync_write_accepted_countNumber of SyncWrites accepted
vb_replica_sync_write_committed_countNumber of SyncWrites committed
vb_replica_sync_write_aborted_countNumber of SyncWrites aborted
vb_replica_hp_vb_req_sizeNum of async high priority requests

2.3.3 Pending vBucket stats

StatDescription
vb_pending_numNumber of pending vBuckets
vb_pending_curr_itemsNumber of pending non-deleted items
vb_pending_num_non_residentNumber of non-resident items
vb_pending_perc_mem_resident% memory resident
vb_pending_ejectNumber of times item values got ejected
vb_pending_expiredNumber of times an item was expired
vb_pending_ht_memoryMemory overhead of the hashtable
vb_pending_ht_memory_overheadMemory overhead of the hashtable
vb_pending_itm_memoryTotal memory of all items in pending
vBuckets (StoredValue + key + value Blob)
vb_pending_meta_data_memoryMetadata memory of all items in pending
vBuckets (StoredValue + key)
vb_pending_meta_data_diskTotal metadata disk
vb_pending_checkpoint_memoryMemory of pending items in all checkpoints
vb_pending_checkpoint_memory_overheadMemory of all pending checkpoints structures
vb_pending_ops_createNumber of create operations
vb_pending_ops_updateNumber of update operations
vb_pending_ops_deleteNumber of delete operations
vb_pending_ops_rejectNumber of rejected operations
vb_pending_queue_sizePending items in disk queue
vb_pending_backfill_queue_sizeItems in pending vbucket backfill queue
vb_pending_queue_memoryMemory used for disk queue
vb_pending_queue_ageSum of disk queue item age in milliseconds
vb_pending_queue_pendingTotal bytes of pending writes
vb_pending_queue_fillTotal enqueued items
vb_pending_queue_drainTotal drained items
vb_pending_rollback_item_countNum of items rolled back
vb_pending_hp_vb_req_sizeNum of async high priority requests

2.4 vBucket detail stats

The stats below are listed for each vbucket.

StatDescription
num_itemsNumber of items in this vbucket
num_tmp_itemsNumber of temporary items in memory
num_non_residentNumber of non-resident items
vb_pending_perc_mem_resident% memory resident
vb_pending_ejectNumber of times item values got ejected
vb_pending_expiredNumber of times an item was expired
ht_memoryMemory overhead of the hashtable
ht_num_deleted_itemsNumber of deleted items in the hashtable
ht_num_in_memory_itemsNumber of in-memory items in the hashtable
ht_num_in_memory_non_resident_itemsNumber of in-memory non-resident items (i.e. items which only have their metadata in memory)
ht_num_itemsNumber of items in the hashtable
ht_num_temp_itemsNumber of temporary items in the hashable
ht_item_memoryTotal item memory
ht_cache_sizeTotal size of cache (Includes non resident
items)
num_ejectsNumber of times an item was ejected from
memory
ops_createNumber of create operations
ops_updateNumber of update operations
ops_deleteNumber of delete operations
ops_rejectNumber of rejected operations
queue_sizePending items in disk queue
backfill_queue_sizeItems in backfill queue
queue_memoryMemory used for disk queue
queue_ageSum of disk queue item age in milliseconds
queue_fillTotal enqueued items
queue_drainTotal drained items
pending writesTotal bytes of pending writes
db_data_sizeTotal size of valid data on disk
db_file_sizeTotal size of the db file
db_prepare_sizeTotal size of SyncWrite prepares on disk
high_seqnoThe last seqno assigned by this vbucket
purge_seqnoThe last seqno purged by the compactor
bloom_filterStatus of the vbucket’s bloom filter
bloom_filter_sizeSize of the bloom filter bit array
bloom_filter_key_countNumber of keys inserted into the bloom
filter, considers overlapped items as one,
so this may not be accurate at times.
uuidThe current vbucket uuid
rollback_item_countNum of items rolled back
hp_vb_req_sizeNum of async high priority requests
max_casMaximum CAS of all items in the vbucket.
This is a hybrid logical clock value in
nanoseconds.
max_cas_strmax_cas as a time stamp string (seconds
since epoch).
total_abs_driftThe accumulated absolute drift for this
vbucket’s hybrid logical clock in
microseconds.
total_abs_drift_countThe number of updates applied to
total_abs_drift.
drift_ahead_threshold_exceededThe number of HLC updates that had a value
ahead of the local HLC and were over the
drift_ahead_threshold.
drift_ahead_thresholdThe ahead threshold in ns.
drift_behind_threshold_exceededThe number of HLC updates that had a value
behind the local HLC and were over the
drift_behind_threshold.
drift_behind_thresholdThe behind threshold in ns.
logical_clock_ticksHow many times this vbucket’s HLC has
returned logical clock ticks.
might_contain_xattrsTrue if the vbucket might contain xattrs.
True means that Xattrs were stored to the
vbucket, note that the flag does not clear
itself if all xattrs were removed.
high_prepared_seqnoDurability: The seqno of the highest
prepared mutation the vbucket is tracking
high_completed_seqnoDurability: The seqno of the highest
durable write that has completed, completed
includes both committed and aborted writes.

For Ephemeral buckets, the following additional statistics are listed for each vbucket:

StatDescription
seqlist_countnumber of documents in this VBucket’s sequence list.
seqlist_deleted_countCount of deleted documents in this VBucket’s sequence list.
seqlist_high_seqnoHigh sequence number in sequence list for this VBucket.
seqlist_highest_deduped_seqnoHighest de-duplicated sequence number in sequence list for this VBucket.
seqlist_read_range_beginStarting sequence number for this VBucket’s sequence list read range. Marks the lower bound of possible stale documents in the sequence list.
seqlist_read_range_endEnding sequence number for this VBucket’s sequence list read range. Marks the upper bound of possible stale documents in the sequence list.
seqlist_read_range_countCount of elements for this VBucket’s sequence list read range (i.e. end - begin).
seqlist_stale_countCount of stale documents in this VBucket’s sequence list.
seqlist_stale_value_bytesNumber of bytes of stale values in this VBucket’s sequence list.
seqlist_stale_metadata_bytesNumber of bytes of stale metadata (key + fixed metadata) in this VBucket’s sequence list.

2.5 vBucket seqno stats

StatsDescription
——————————+——————————————–
abs_high_seqnoThe last seqno assigned by this vbucket
high_seqnoThe last seqno assigned by this vbucket, in
in case of replica, the last closed check-
point’s end seqno.
last_persisted_seqnoThe last persisted seqno for the vbucket
purge_seqnoThe last seqno purged by the compactor
uuidThe current vbucket uuid
last_persisted_snap_startThe last persisted snapshot start seqno for
the vbucket
last_persisted_snap_endThe last persisted snapshot end seqno for
the vbucket

2.6 vBucket failover stats

StatsDescription
——————————+——————————————–
num_entriesNumber of entries in the failover table of
this vbucket
erroneous_entries_erasedNumber of erroneous entries erased in the
failover table of this vbucket
n:idvb_uuid of nth failover entry in the
failover table of this vbucket
n:seqseqno of nth failover entry in the
failover table of this vbucket

2.7 Dcp Stats

Each stat begins with ep_dcpq: followed by a unique client_id and another colon. For example, if your client is named, slave1, the created stat would be ep_dcpq:slave1:created.

***Consumer Connections

createdCreation time for the tap connection
pending_disconnectTrue if we’re hanging up on this client
reservedTrue if the dcp stream is reserved
supports_ackTrue if the connection use flow control
total_acked_bytesThe amount of bytes that the consumer has acked
unacked_bytesThe amount of bytes the consumer has processed but not acked
typeThe connection type (producer or consumer)
max_buffer_bytesSize of flow control buffer
pausedtrue if this client is blocked
paused_reasonDescription of why client is paused

****Per Stream Stats

buffer_bytesThe amount of unprocessed bytes
buffer_itemsThe amount of unprocessed items
end_seqnoThe seqno where this stream should end
flagsThe flags used to create this stream
items_readyWhether the stream has messages ready to send
ready_queue_memoryMemory occupied by elements in the DCP readyQ
opaqueThe unique stream identifier
snap_end_seqnoThe start seqno of the last snapshot received
snap_start_seqnoThe end seqno of the last snapshot received
start_seqnoThe start start seqno used to create this stream
stateThe stream state (pending, reading, or dead)
vb_uuidThe vb uuid used to create this stream

***Producer Connections

buf_backfill_bytesThe amount of bytes backfilled but not sent
buf_backfill_itemsThe amount of items backfilled but not sent
bytes_sentThe amount of unacked bytes sent to the consumer
createdCreation time for the tap connection
flow_controlTrue if the connection use flow control
items_remainingThe amount of items remaining to be sent
items_sentThe amount of items already sent to the consumer
last_sent_timeThe last time this connection sent a message
last_receive_timeThe last time this connection received a message
max_buffer_bytesThe maximum amount of bytes that can be sent without
receiving an ack from the consumer
noop_enabledWhether or not this connection sends noops
noop_tx_intervalThe time interval between noop messages
noop_waitWhether or not this connection is waiting for a
noop response from the consumer
pending_disconnectTrue if we’re hanging up on this client
priorityThe connection priority for streaming data
num_streamsTotal number of streams in the connection in any state
num_dead_streamsTotal number of dead streams in the connection
reservedTrue if the dcp stream is reserved
supports_ackTrue if the connection use flow control
total_acked_bytesThe amount of bytes that have been acked by the
consumer when flow control is enabled
total_bytes_sentThe amount of bytes actually sent to the consumer
total_uncompressed_data_sizeSize of data before compression sent to the consumer.
Only present if compression is enabled
typeThe connection type (producer or consumer)
unacked_bytesThe amount of bytes the consumer has no acked
backfill_num_activeNumber of active (running) backfills
backfill_num_snoozingNumber of snoozing (running) backfills
backfill_num_pendingNumber of pending (not running) backfills
backfill_orderOrder backfills should be scheduled
pausedtrue if this client is blocked
paused_reasonDescription of why client is paused
send_stream_end_on_client_close_streamSend STREAM_END msg when DCP client closes stream

****Per Stream Stats

backfill_disk_itemsThe amount of items read during backfill from disk
backfill_mem_itemsThe amount of items read during backfill from memory
backfill_sentThe amount of items sent to the consumer during the
end_seqnoThe seqno send mutations up to
flagsThe flags supplied in the stream request
items_readyWhether the stream has items ready to send
last_sent_seqnoThe last seqno sent by this stream
last_sent_snap_end_seqnoThe last snapshot end seqno sent by active stream
last_read_seqnoThe last seqno read by this stream from disk or memory
ready_queue_memoryMemory occupied by elements in the DCP readyQ
memory_phaseThe amount of items sent during the memory phase
opaqueThe unique stream identifier
snap_end_seqnoThe last snapshot end seqno (Used if a consumer is
resuming a stream)
snap_start_seqnoThe last snapshot start seqno (Used if a consumer is
resuming a stream)
start_seqnoThe seqno to start sending mutations from
stateThe stream state (pending, backfilling, in-memory,
takeover-send, takeover-wait, or dead)
vb_uuidThe vb uuid used in the stream request
cur_snapshot_typeThe type of the current snapshot being received
cur_snapshot_startThe start seqno of the current snapshot being
received
cur_snapshot_endThe end seqno of the current snapshot being received

2.8 Dcp Aggregated Stats

Aggregated dcp stats allow dcp connections to be logically grouped and aggregated together by prefixes.

For example, if all of your dcp connections started with xdcr: or replication, you could call stats dcpagg : to request stats grouped by everything before the first : character, giving you a set for xdcr and a set for replication.

2.8.1 Results

[prefix]:countNumber of connections matching this prefix
[prefix]:producer_countTotal producer connections with this prefix
[prefix]:items_sentTotal items sent with this prefix
[prefix]:items_remainingTotal items remaining to be sent with this
prefix
[prefix]:total_bytesTotal number of bytes sent with this prefix
[prefix]:total_uncompressed_data_sizeSize of data before compression sent to the
consumer with this prefix. Only present if
compression is enabled
[prefix]:backoffTotal number of backoff events

2.9 Dcp ConnMap Stats

ep_dcp_num_running_backfillsTotal number of running backfills across all
dcp connections
ep_dcp_max_running_backfillsMax running backfills we can have across all
dcp connections
ep_dcp_dead_conn_countTotal dead connections

2.10 Timing Stats

Timing stats provide histogram data from high resolution timers over various operations within the system.

2.10.1 General Form

As this data is multi-dimensional, some parsing may be required for machine processing. It’s somewhat human readable, but the stats script mentioned in the Getting Started section above will do fancier formatting for you.

Consider the following sample stats:

STAT disk_insert_8,16 9488
STAT disk_insert_16,32 290
STAT disk_insert_32,64 73
STAT disk_insert_64,128 86
STAT disk_insert_128,256 48
STAT disk_insert_256,512 2
STAT disk_insert_512,1024 12
STAT disk_insert_1024,2048 1

This tells you that disk_insert took 8-16µs 9,488 times, 16-32µs 290 times, and so on.

The same stats displayed through the stats CLI tool would look like this:

disk_insert (10008 total)
   8us - 16us    : ( 94.80%) 9488 ###########################################
   16us - 32us   : ( 97.70%)  290 #
   32us - 64us   : ( 98.43%)   73
   64us - 128us  : ( 99.29%)   86
   128us - 256us : ( 99.77%)   48
   256us - 512us : ( 99.79%)    2
   512us - 1ms   : ( 99.91%)   12
   1ms - 2ms     : ( 99.92%)    1

2.10.2 Available Stats

The following histograms are available from “timings” in the above form to describe when time was spent doing various things:

bg_waitbg fetches waiting in the dispatcher queue
bg_loadbg fetches waiting for disk
set_with_metaset_with_meta latencies
access_scanneraccess scanner run times
checkpoint_removercheckpoint remover run times
item_pageritem pager run times
expiry_pagerexpiry pager run times
pending_opsclient connections blocked for operations
in pending vbuckets
storage_ageAnalogous to ep_storage_age in main stats
data_ageAnalogous to ep_data_age in main stats
get_cmdservicing get requests
arith_cmdservicing incr/decr requests
get_stats_cmdservicing get_stats requests
get_vb_cmdservicing vbucket status requests
set_vb_cmdservicing vbucket set state commands
del_vb_cmdservicing vbucket deletion commands
chk_persistence_cmdwaiting for checkpoint persistence
notify_iowaking blocked connections
paged_out_timetime (in seconds) objects are non-resident
disk_insertwaiting for disk to store a new item
disk_updatewaiting for disk to modify an existing item
disk_delwaiting for disk to delete an item
disk_vb_delwaiting for disk to delete a vbucket
disk_commitwaiting for a commit after a batch of updates
item_alloc_sizesItem allocation size counters (in bytes)
bg_batch_sizeBatch size for background fetches
persistence_cursor_get_all_itemsTime spent in fetching all items by
persistence cursor from checkpoint queues
dcp_cursors_get_all_itemsTime spent in fetching all items by all dcp
cursors from checkpoint queues
sync_write_commit_majorityCommit duration for level=majority SyncWrites
sync_write_commit_majority_and_persist_on_masterCommit duration for level=majorityPersistActive SyncWrites
sync_write_commit_persist_to_majorityCommit duration for level=persistMajority SyncWrites

The following histograms are available from “eviction” and provide a histogram of execution frequencies and eviction thresholds. Note, these statstics are only valid for the hifi_mfu eviction policy.

ep_active_or_pending_frequency_values_evictedProbabilistic count of frequencies
that were evicted
ep_replica_frequency_values_evictedProbabilistic count of frequencies
that were evicted
ep_active_or_pending_frequency_values_snapshotSnapshot of last frequency histogram
ep_replica_frequency_values_snapshotSnapshot of last frequency histogram

The following histograms are available from “scheduler” and “runtimes” describing the scheduling overhead times and task runtimes incurred by various IO and Non-IO tasks respectively:

READ tasks
bg_fetcher_taskshistogram of scheduling overhead/task
runtimes for background fetch tasks
bg_fetcher_meta_taskshistogram of scheduling overhead/task
runtimes for background fetch meta tasks
vkey_stat_bg_fetcher_taskshistogram of scheduling overhead/task
runtimes for fetching item from disk for
vkey stat tasks
warmup_taskshistogram of scheduling overhead/task
runtimes for warmup tasks
WRITE tasks
vbucket_persist_high_taskshistogram of scheduling overhead/task
runtimes for snapshot vbucket state in
high priority tasks
vbucket_persist_low_taskshistogram of scheduling overhead/task
runtimes for snapshot vbucket state in
low priority tasks
vbucket_deletion_taskshistogram of scheduling overhead/task
runtimes for vbucket deletion tasks
flusher_taskshistogram of scheduling overhead/task
runtimes for flusher tasks
flush_all_taskshistogram of scheduling overhead/task
runtimes for flush all tasks
compactor_taskshistogram of scheduling overhead/task
runtimes for vbucket level compaction
tasks
statsnap_taskshistogram of scheduling overhead/task
runtimes for stats snapshot tasks
mutation_log_compactor_taskshistogram of scheduling overhead/task
runtimes for access log compaction tasks
AUXIO tasks
access_scanner_taskshistogram of scheduling overhead/task
runtimes for access scanner tasks
backfill_taskshistogram of scheduling overhead/task
runtimes for backfill tasks
NONIO tasks
conn_notification_taskshistogram of scheduling overhead/task
runtimes for connection notification
tasks
checkpoint_remover_taskshistogram of scheduling overhead/task
runtimes for checkpoint removal tasks
vb_memory_deletion_taskshistogram of scheduling overhead/task
runtimes for memory deletion of vbucket
tasks
checkpoint_stats_taskshistogram of scheduling overhead/task
runtimes for checkpoint stats tasks
item_pager_taskshistogram of scheduling overhead/task
runtimes for item pager tasks
hashtable_resize_taskshistogram of scheduling overhead/task
runtimes for hash table resizer tasks
pending_ops_taskshistogram of scheduling overhead/task
runtimes for processing dcp bufferred
items tasks
conn_manager_taskshistogram of scheduling overhead/task
runtimes for dcp/tap connection manager
tasks
defragmenter_taskshistogram of scheduling overhead/task
runtimes for the in-memory defragmenter
tasks
workload_monitor_taskshistogram of scheduling overhead/task
runtimes for the workload monitor which
detects and sets the workload pattern

2.11 Hash Stats

Hash stats provide information on your vbucket hash tables.

Requesting these stats does affect performance, so don’t do it too regularly, but it’s useful for debugging certain types of performance issues. For example, if your hash table is tuned to have too few buckets for the data load within it, the max_depth will be too large and performance will suffer.

avg_countThe average number of items per vbucket
avg_maxThe average max depth of a vbucket hash table
avg_minThe average min depth of a vbucket hash table
largest_maxThe largest hash table depth of in all vbuckets
largest_minThe the largest minimum hash table depth of all vbuckets
max_countThe largest number of items in a vbucket
min_countThe smallest number of items in a vbucket
total_countsThe total numer of items in all vbuckets

It is also possible to get more detailed hash tables stats by using ‘hash detail’. This will print per-vbucket stats.

Each stat is prefixed with vb_ followed by a number, a colon, then the individual stat name.

For example, the stat representing the size of the hash table for vbucket 0 is vb_0:size.

stateThe current state of this vbucket
sizeNumber of hash buckets
locksNumber of locks covering hash table operations
min_depthMinimum number of items found in a bucket
max_depthMaximum number of items found in a bucket
reportedNumber of items this hash table reports having
countedNumber of items found while walking the table
resizedNumber of times the hash table resized
mem_sizeRunning sum of memory used by each item
mem_size_countedCounted sum of current memory used by each item

2.12 Checkpoint Stats

Checkpoint stats provide detailed information on per-vbucket checkpoint datastructure.

Like Hash stats, requesting these stats has some impact on performance. Therefore, please do not poll them from the server frequently. Each stat is prefixed with vb_ followed by a number, a colon, and then each stat name.

cursor_name:cursor_checkpoint_idCheckpoint ID at which the cursor is
name ‘cursor_name’ is pointing now
cursor_name:cursor_distanceThe distance of cursor from checkpoint
begin
cursor_name:cursor_seqnoThe seqno at which the cursor
‘cursor_name’ is pointing now
cursor_name:cursor_opThe type of operation of the item pointed
by cursor
cursor_name:num_visitsNumber of times a batch of items have been
drained from a checkpoint of ‘cursor_name’
cursor_name:num_items_for_cursorNumber of items remaining for the cursor
open_checkpoint_idID of the current open checkpoint
num_conn_cursorsNumber of referencing dcp/tap cursors
num_checkpoint_itemsNumber of total items in a checkpoint
datastructure
num_open_checkpoint_itemsNumber of items in the open checkpoint
(empty item excluded)
num_checkpointsNumber of all checkpoints in the bucket,
including all Vbuckets/CMs/Destroyers
num_checkpoints_pending_destructionNumber of checkpoints detached from CMs
and owned by Destroyers
stateThe state of the vbucket this checkpoint
contains data for
persisted_checkpoint_idThe last persisted checkpoint number
mem_usageTotal memory taken up by items in all
checkpoints under given manager

Additionally each Checkpoint will generate the following stats, these are prefixed with the vbucket and the id of the Checkpoint, e.g. “vb_0:id_52:state”

stateCheckpoint open or closed
typeType of checkpoint, disk or memory
key_index_allocator_bytesThe number of bytes currently allocated to
the key index(s) as returned by the
underlying std::allocator implementation,
including keys.
to_write_allocator_bytesThe number of bytes currently allocated to
the toWrite queue as returned by the
underlying std::allocator implementation
mem_usage_queued_itemsSize of all items queued in checkpoints,
computed by checkpoint counters
mem_usage_queue_overheadBytes consumed by the toWrite struct
internals, computed by checkpoint counters
mem_usage_key_index_overheadBytes consumed by the key index. Accounts
both struct internals and keys. Computed
by checkpoint counters
num_itemsNumber of items queued in the checkpoint
(empty item excluded)

2.13 Memory Stats

This provides various memory-related stats including some stats from jemalloc.

mem_usedEngine’s total memory usage
mem_used_estimateEngine’s total estimated memory usage
This is a faster stat to read, but
lags mem_used as it’s only updated
when a threshold is crossed see
mem_used_merge_threshold
mem_used_merge_thresholdA threshold which triggers the merge
of per-core memory used into mem_used
bytesEngine’s total memory usage
ep_kv_sizeMemory used to store item metadata,
keys and values, no matter the
vbucket’s state. If an item’s value
is ejected, this stat will be
decremented by the size of the
item’s value.
ep_value_sizeMemory used to store values for
resident keys
ep_overheadExtra memory used by transient data
like persistence queue, replication
queues, checkpoints, etc
ep_max_sizeMax amount of data allowed in memory
ep_mem_low_watLow water mark for auto-evictions
ep_mem_low_wat_percentLow water mark (as a percentage)
ep_mem_high_watHigh water mark for auto-evictions
ep_mem_high_wat_percentHigh water mark (as a percentage)
ep_oom_errorsNumber of times unrecoverable OOMs
happened while processing operations
ep_tmp_oom_errorsNumber of times temporary OOMs
happened while processing operations
ep_blob_numThe number of blob objects in the
cache
ep_blob_overheadThe “unused” memory caused by the
allocator returning bigger chunks
than requested
ep_storedval_sizeMemory used by storedval objects
ep_storedval_overheadThe “unused” memory caused by the
allocator returning bigger chunks
than requested
ep_storedval_numThe number of storedval objects
allocated
ep_item_numThe number of item objects allocated
ep_arena_memory_allocatedThe total memory allocated from the
engine’s arena (same as
ep_arena:allocated below)
ep_arena_memory_residentThe resident set size of the engine’s
arena.

The following stats are found by querying jemalloc, definitions of the jemalloc stats can be found at:

ep_arena:allocated:ep_arena:small.allocated + ep_arena:large.allocated
ep_arena:arena:The id of the arena registered to the bucket
ep_arena:base:This is “stats.arenas.<i>.base” from jemalloc where <i> is the bucket’s arena
ep_arena:fragmentation_size:ep_arena:resident - ep_arena:allocated
ep_arena:internal:This is “stats.arenas.<i>.internal” from jemalloc where <i> is the bucket’s arena
ep_arena:large.allocated:This is “stats.arenas.<i>.large.allocate” from jemalloc where <i> is the bucket’s arena
ep_arena:mapped:This is “stats.arenas.<i>.mapped” from jemalloc where <i> is the bucket’s arena
ep_arena:resident:This is “stats.arenas.<i>.resident” from jemalloc where <i> is the bucket’s arena
ep_arena:retained:This is “stats.arenas.<i>.retained” from jemalloc where <i> is the bucket’s arena
ep_arena:small.allocated:This is “stats.arenas.<i>.small.allocated” from jemalloc where <i> is the bucket’s arena
ep_arena_global:allocated:ep_arena_global:small.allocated + ep_arena_global:large.allocated
ep_arena_global:arena:The id of the arena used for global (non bucket) allocations.
ep_arena_global:base:See “ep_arena:” entry, this is the stat query but for the ‘global’ arena.
ep_arena_global:fragmentation_size:ep_arena_global:resident - ep_arena_global:allocated
ep_arena_global:internal:See “ep_arena:” entry, this is the stat query but for the ‘global’ arena.
ep_arena_global:large.allocated:See “ep_arena:” entry, this is the stat query but for the ‘global’ arena.
ep_arena_global:mapped:See “ep_arena:” entry, this is the stat query but for the ‘global’ arena.
ep_arena_global:resident:See “ep_arena:” entry, this is the stat query but for the ‘global’ arena.
ep_arena_global:retained:See “ep_arena:” entry, this is the stat query but for the ‘global’ arena.
ep_arena_global:small.allocated:See “ep_arena:” entry, this is the stat query but for the ‘global’ arena.

2.14 Stats Key and Vkey

key_casThe keys current cas valueKV
key_exptimeExpiration time from the epochKV
key_flagsFlags for this keyKV
key_is_dirtyIf the value is not yet persistedKV
key_is_residentIf the value is resident in memoryKV
key_validSee description belowV
key_vb_stateThe vbucket state of this keyKV

All of the above numeric statistics (cas, exptime, flags) are printed as decimal integers.

key_valid can have the following responses:

this_is_a_bug - Some case we didn’t take care of. dirty - The value in memory has not been persisted yet. length_mismatch - The key length in memory doesn’t match the length on disk. data_mismatch - The data in memroy doesn’t match the data on disk. flags_mismatch - The flags in memory don’t match the flags on disk. valid - The key is both on disk and in memory ram_but_not_disk - The value doesn’t exist yet on disk. item_deleted - The item has been deleted.

2.15 Warmup

Stats warmup shows statistics related to warmup logic

ep_warmupShows if warmup is enabled / disabled
ep_warmup_estimated_key_countEstimated number of keys in database
ep_warmup_estimated_value_countEstimated number of values in database
ep_warmup_stateThe current state of the warmup thread
ep_warmup_threadWarmup thread status
ep_warmup_key_countNumber of keys warmed up
ep_warmup_value_countNumber of values warmed up
ep_warmup_dupsDuplicates encountered during warmup
ep_warmup_oomOOMs encountered during warmup
ep_warmup_timeTime (µs) spent by warming data
ep_warmup_keys_timeTime (µs) spent by warming keys
ep_warmup_mutation_logNumber of keys present in mutation log
ep_warmup_access_logNumber of keys present in access log
ep_warmup_min_items_thresholdPercentage of total items warmed up
before we enable traffic
ep_warmup_min_memory_thresholdPercentage of max mem warmed up before
we enable traffic

2.16 KV Store Stats

These provide various low-level stats and timings from the underlying KV storage system and useful to understand various states of the storage system.

The following stats are available for all database engine:

openNumber of database open operations
closeNumber of database close operations
readTimeTime spent in read operations
readSizeSize of data in read operations
writeTimeTime spent in write operations
writeSizeSize of data in write operations
deleteTime spent in delete() calls

The following stats are available for the CouchStore database engine:

backend_typeType of backend database engine
commitTime spent in CouchStore commit operation
compactionTime spent in compacting vbucket database file
numLoadedVbNumber of Vbuckets loaded into memory
lastCommDocsNumber of docs in the last commit
failure_setNumber of failed set operation
failure_getNumber of failed get operation
failure_vbsetNumber of failed vbucket set operation
save_documentsTime spent in CouchStore save documents operation
io_bg_fetch_docs_readNumber of documents (full and meta-only) fetched from disk
io_bg_fetch_doc_bytesNumber of bytes read while fetching documents (key + value + rev_meta)
io_flusher_write_amplificationNumber of bytes written to disk during front-end flushing, divided by the document bytes for each document saved (key + metadata + value).
io_total_write_amplificationNumber of bytes written to disk during front-end flushing and compaction, divided by the document bytes for each document saved (key + metadata + value).
io_num_writeNumber of io write operations
io_document_write_bytesNumber of document bytes written (key + value + rev_meta)
io_total_read_bytesNumber of bytes read (total, including Couchstore B-Tree and other overheads)
io_total_write_bytesNumber of bytes written (total, including Couchstore B-Tree and other overheads)
io_compaction_read_bytesNumber of bytes read (compaction only, includes Couchstore B-Tree and other overheads)
io_compaction_write_bytesNumber of bytes written (compaction only, includes Couchstore B-Tree and other overheads)
block_cache_hitsNumber of block cache hits in buffer cache provided by underlying store
block_cache_missesNumber of block cache misses in buffer cache provided by underlying store
getMultiFsReadCountNumber of filesystem read()s per getMulti() request
getMultiFsReadPerDocCountNumber of filesystem read()s per getMulti() request, divided by the number of documents fetched; gives an average read() count per fetched document

2.17 KV Store Timing Stats

KV Store Timing stats provide timing information from the underlying storage system. These stats are on shard (group of partitions) level.

2.17.1 Available Stats

The following histograms are available from “kvtimings” in the form described in Timings section above. These stats are prefixed with the rw_<Shard number>: indicating the times spent doing various things:

committime spent in commit operations
compacttime spent in file compaction operations
snapshottime spent in VB state snapshot operations
deletetime spent in delete operations
save_documentstime spent in persisting documents in storage
readTimeTime spent in read operations, measured from when the read was initially requsted (bgFetch queued), until when the KVStore completes the read of that document.
readSizeSize of data in read operations
writeTimetime spent in writing to storage subsystem
writeSizesizes of writes given to storage subsystem
saveDocCountbatch sizes of the save documents calls
fsReadTimetime spent in doing filesystem reads
fsWriteTimetime spent in doing filesystem writes
fsSyncTimetime spent in doing filesystem sync operations
fsReadSizesizes of various filesystem reads issued
fsWriteSizesizes of various filesystem writes issued
fsReadSeekvalues of various seek operations in file
flusherWriteAmplificationRatioWrite Amplification per saveDocs batch

2.18 Workload Raw Stats

Some information about the number of shards and Executor pool information. These are available as “workload” stats:

ep_workload:num_shardsnumber of shards or groups of partitions
ep_workload:num_writersnumber of threads that prioritize write ops
ep_workload:num_readersnumber of threads that prioritize read ops
ep_workload:num_auxionumber of threads that prioritize aux io ops
ep_workload:num_nonionumber of threads that prioritize non io ops
ep_workload:num_sleepersnumber of threads that are sleeping
ep_workload:ready_tasksnumber of global tasks that are ready to run

Additionally the following stats on the current state of the TaskQueues are also presented

HiPrioQ_Writer:InQsizecount high priority bucket writer tasks waiting
HiPrioQ_Writer:OutQsizecount high priority bucket writer tasks runnable
HiPrioQ_Reader:InQsizecount high priority bucket reader tasks waiting
HiPrioQ_Reader:OutQsizecount high priority bucket reader tasks runnable
HiPrioQ_AuxIO:InQsizecount high priority bucket auxio tasks waiting
HiPrioQ_AuxIO:OutQsizecount high priority bucket auxio tasks runnable
HiPrioQ_NonIO:InQsizecount high priority bucket nonio tasks waiting
HiPrioQ_NonIO:OutQsizecount high priority bucket nonio tasks runnable
LowPrioQ_Writer:InQsizecount low priority bucket writer tasks waiting
LowPrioQ_Writer:OutQsizecount low priority bucket writer tasks runnable
LowPrioQ_Reader:InQsizecount low priority bucket reader tasks waiting
LowPrioQ_Reader:OutQsizecount low priority bucket reader tasks runnable
LowPrioQ_AuxIO:InQsizecount low priority bucket auxio tasks waiting
LowPrioQ_AuxIO:OutQsizecount low priority bucket auxio tasks runnable
LowPrioQ_NonIO:InQsizecount low priority bucket nonio tasks waiting
LowPrioQ_NonIO:OutQsizecount low priority bucket nonio tasks runnable

2.19 Dispatcher Stats/JobLogs

This provides the stats from AUX dispatcher and non-IO dispatcher, and from all the reader and writer threads running for the specific bucket. Along with stats, the job logs for each of the dispatchers and worker threads is also made available.

The following stats are available for the workers and dispatchers:

stateThreads’s current status: running, sleeping etc.
runtimeThe amount of time since the thread started running
taskThe activity/job the thread is involved with at the moment

The following stats are for individual job logs:

starttimeThe timestamp when the job started
runtimeTime it took for the job to run
taskThe activity/job the thread ran during that time

2.20 Scope and Collection stats

Values for scopes and collections are available from a number of keys. The entire set of scopes/collection or individual scope or collection can be interrogated using name or id.

Available keys:

3 scopes [scope_name]

  • Stats for all scopes or a single scope (using scope name as a key)

4 collections [scope_name.collection_name]

  • Stats for all collections or a single collection (using collection name as a key)

5 scopes-byid id

  • Stats for a single scope using the id as a key

6 collections-byid id

  • Stats for a single collection using the id as a key

Further details are available at a vbucket granularity, individual vbucket view is an optional argument.

7 scopes-details [vbucket]

8 collections-details [vbucket]

8.0.1 Collections

“collections” and “collections-byid” returns the following statistics, most keys returned are prefixed with the scope-id and collection-id encoded as 0x prefixed hexadecimal. For brevity, ‘sid’ and ‘cid’ are used for scope-id and collection-id.

sid:cid:disk_sizeApproximate disk-usage of the collection. Note the sum of all collection disk-sizes does not equal the bucket disk usage
sid:cid:itemsNumber of items stored in the collection.
sid:cid:maxTTLThe Time-To-Live value for the collection, omitted if none defined.
sid:cid:mem_usedApproximate memory-usage of the collection. Note the sum of all collection mem_used does not equal the bucket mem_used.
sid:cid:nameThe collection’s name.
sid:cid:ops_deleteThe number of delete operations performed against the collection.
sid:cid:ops_getThe number of get operations performed against the collection.
sid:cid:ops_storeThe number of storage operations performed against the collection.
sid:cid:scope_nameThe name of the collection’s scope.
manifest_uidThe uid of the last manifest accepted from the cluster, only returned when all collections are requested (no name or id provided)

Note for disk-size and upgrade: An upgrade to ‘cheshire-cat’ means all existing data becomes owned by the _default collection, if the upgrade was off-line, the disk-size is initialised to the total disk used by the bucket.

“collections-details” returns vbucket collection data, an optional vbucket (decimal value) allows a single vbucket to be inspected. Keys returned are prefixed with the vbucket ID as “vb_x”, where x is a decimal value and may also include the collection-id encoded as a 0x hexadecimal value (cid used in table).

vb_x:cid:high_seqnoThe high-seqno of the collection.
vb_x:cid:itemsThe number of items the collection stores in this vbucket.
vb_x:cid:persisted_high_seqnoThe highest persisted seqno.
vb_x:cid:scopeThe collection’s scope (as an 0x id).
vb_x:cid:maxTTLThe Time-To-Live value for the collection, omitted if none defined.
vb_x:cid:start_seqnoThe start seqno of the collection, the seqno when it was created.
vb_x:collectionsThe number of collections the vbucket knows about.
vb_x:manifest_uidThe id of the manifest last used to update the vbucket.

8.0.2 Scopes

“scopes” and “scopes-byid” returns the following statistics. For stats related to the scope only, they are prefixed with the scope-id as a 0x prefixed hexadecimal value, for collections within a scope they are prefixed with scope-id and collection-id as a 0x prefixed hexadecimal value. For brevity, ‘sid’ and ‘cid’ are used for scope-id and collection-id.

When a specific scope is selected, each collection within the scope is returned. When the scope (no argument) key is used only the names of the collections in each scope are returned. The sid:cid stats returned within the scopes view are the same values (and definitions) as the keys/value returned from “collections” and “collections-byid”.

sid:cid:nameThe name of a collection in the scope, multiple names maybe returned.
sid:collectionsThe count of collections in the scope.
sid:disk_sizeThe sum of all collection ‘disk_size’.
sid:itemsThe sum of all collection ‘items’.
sid:mem_usedThe sum of all collection ‘mem_used’.
sid:nameThe name of the scope.
sid:ops_deleteThe sum of all collection ‘ops_delete’.
sid:ops_getThe sum of all collection ‘ops_get’.
sid:ops_storeThe sum of all collection ‘ops_store.
manifest_uidThe uid of the last manifest accepted from the cluster, only returned when all scopes are requested (no name or id provided)

“scopes-details” returns vbucket scope data, an optional vbucket (decimal value) allows a single vbucket to be inspected. Keys returned are prefixed with the vbucket ID as “vb_x”, where x is a decimal value and may also include the scope/collection-id encoded as a 0x hexadecimal value (sid/cid used in table).

vb_x:scopesThe number of scopes.
vb_x:sidAll of the known scope-ids returned, the value is the index position from the internal container
vb_x:sid:cid:itemsThe item count of a collection, repeated for all collections.
vb_x:manifest_uidThe id of the manifest last used to update the vbucket.

8.1 Stats Reset

Resets the list of stats below.

Reset Stats:

ep_bg_load
ep_bg_wait
ep_bg_max_load
ep_bg_min_load
ep_bg_max_wait
ep_bg_min_wait
ep_commit_time
ep_flush_duration
ep_flush_duration_highwat
ep_io_bg_fetch_docs_read
ep_io_num_write
ep_io_bg_fetch_doc_bytes
ep_io_write_bytes
ep_items_expelled_from_checkpoints
ep_items_rm_from_checkpoints
ep_num_eject_failures
ep_num_pager_runs
ep_num_not_my_vbuckets
ep_num_value_ejects
ep_pending_ops_max
ep_pending_ops_max_duration
ep_pending_ops_total
ep_vbucket_del_max_walltime
pending_ops

Reset Histograms:

bg_load
bg_wait
chk_persistence_cmd
data_age
del_vb_cmd
disk_insert
disk_update
disk_del
disk_vb_del
disk_commit
get_stats_cmd
item_alloc_sizes
get_vb_cmd
notify_io
pending_ops
persistence_cursor_get_all_items
dcp_cursors_get_all_items
set_vb_cmd
storage_age
ep_active_or_pending_frequency_values_evicted
ep_replica_frequency_values_evicted
ep_active_or_pending_frequency_values_snapshot
ep_replica_frequency_values_snapshot

9 Details

9.1 Ages

The difference between ep_storage_age and ep_data_age is somewhat subtle, but when you consider that a given record may be updated multiple times before hitting persistence, it starts to be clearer.

ep_data_age is how old the data we actually wrote is.

ep_storage_age is how long the object has been waiting to be persisted.

9.2 Warming Up

Opening the data store is broken into three distinct phases:

9.2.1 Initializing

During the initialization phase, the server is not accepting connections or otherwise functional. This is often quick, but in a server crash can take some time to perform recovery of the underlying storage.

This time is made available via the ep_dbinit stat.

9.2.2 Warming Up

After initialization, warmup begins. At this point, the server is capable of taking new writes and responding to reads. However, only records that have been pulled out of the storage or have been updated from other clients will be available for request.

(note that records read from persistence will not overwrite new records captured from the network)

During this phase, ep_warmup_thread will report running and ep_warmed_up will be increasing as records are being read.

9.2.3 Complete

Once complete, ep_warmed_up will stop increasing and ep_warmup_thread will report complete.

10 Uuid

The uuid stats allows clients to check if the unique identifier created and assigned to the bucket when it is created. By looking at this a client can verify that the bucket hasn’t been recreated since it was used.