Skip to main content

Pulsar Metrics

Pulsar exposes metrics in Prometheus format that can be collected and used for monitoring the health of the cluster.

Overview​

The metrics exposed by Pulsar are in Prometheus format. The types of metrics are:

  • Counter: a cumulative metric that represents a single monotonically increasing counter whose value can only increase or be reset to zero on restart.
  • Gauge: a gauge is a metric that represents a single numerical value that can arbitrarily go up and down.
  • Histogram: a histogram samples observations (usually things like request durations or response sizes) and counts them in configurable buckets. The _bucket suffix is the number of observations within a histogram bucket, configured with parameter {le="<upper inclusive bound>"}. The _count suffix is the number of observations, shown as a time series and behaves like a counter. The _sum suffix is the sum of observed values, also shown as a time series and behaves like a counter. These suffixes are together denoted by _* in this doc.
  • Summary: similar to a histogram, a summary samples observations (usually things like request durations and response sizes). While it also provides a total count of observations and a sum of all observed values, it calculates configurable quantiles over a sliding time window.

ZooKeeper​

The ZooKeeper metrics are exposed under "/metrics" at port 8000. You can use a different port by configuring the stats_server_port system property.

Server metrics​

NameTypeDescription
zookeeper_server_znode_countGaugeThe number of z-nodes stored.
zookeeper_server_data_size_bytesGaugeThe total size of all of z-nodes stored.
zookeeper_server_connectionsGaugeThe number of currently opened connections.
zookeeper_server_watches_countGaugeThe number of watchers registered.
zookeeper_server_ephemerals_countGaugeThe number of ephemeral z-nodes.

Request metrics​

NameTypeDescription
zookeeper_server_requestsCounterThe total number of requests received by a particular server.
zookeeper_server_requests_latency_msSummaryThe requests latency calculated in milliseconds.
Available labels: type (write, read).
  • write: the requests that write data to ZooKeeper.
  • read: the requests that read data from ZooKeeper.

BookKeeper​

The BookKeeper metrics are exposed under "/metrics" at port 8000. You can change the port by updating prometheusStatsHttpPort in bookkeeper.conf configuration file.

Server metrics​

NameTypeDescription
bookie_SERVER_STATUSGaugeThe server status for bookie server.
  • 1: the bookie is running in writable mode.
  • 0: the bookie is running in readonly mode.
bookkeeper_server_ADD_ENTRY_countCounterThe total number of ADD_ENTRY requests received at the bookie. The success label is used to distinguish successes and failures.
bookkeeper_server_READ_ENTRY_countCounterThe total number of READ_ENTRY requests received at the bookie. The success label is used to distinguish successes and failures.
bookie_WRITE_BYTESCounterThe total number of bytes written to the bookie.
bookie_READ_BYTESCounterThe total number of bytes read from the bookie.
bookkeeper_server_ADD_ENTRY_REQUESTSummaryThe summary of request latency of ADD_ENTRY requests at the bookie. The success label is used to distinguish successes and failures.
bookkeeper_server_READ_ENTRY_REQUESTSummaryThe summary of request latency of READ_ENTRY requests at the bookie. The success label is used to distinguish successes and failures.

Journal metrics​

NameTypeDescription
bookie_journal_JOURNAL_SYNC_countCounterThe total number of journal fsync operations happening at the bookie. The success label is used to distinguish successes and failures.
bookie_journal_JOURNAL_QUEUE_SIZEGaugeThe total number of requests pending in the journal queue.
bookie_journal_JOURNAL_FORCE_WRITE_QUEUE_SIZEGaugeThe total number of force write (fsync) requests pending in the force-write queue.
bookie_journal_JOURNAL_CB_QUEUE_SIZEGaugeThe total number of callbacks pending in the callback queue.
bookie_journal_JOURNAL_ADD_ENTRYSummaryThe summary of request latency of adding entries to the journal.
bookie_journal_JOURNAL_SYNCSummaryThe summary of fsync latency of syncing data to the journal disk.

Storage metrics​

NameTypeDescription
bookie_ledgers_countGaugeThe total number of ledgers stored in the bookie.
bookie_entries_countGaugeThe total number of entries stored in the bookie.
bookie_write_cache_sizeGaugeThe bookie write cache size (in bytes).
bookie_read_cache_sizeGaugeThe bookie read cache size (in bytes).
bookie_DELETED_LEDGER_COUNTCounterThe total number of ledgers deleted since the bookie has started.
bookie_ledger_writable_dirsGaugeThe number of writable directories in the bookie.

Broker​

The broker metrics are exposed under "/metrics/" at port 8080. You can change the port by updating webServicePort to a different port in broker.conf configuration file.

All the metrics exposed by a broker are labelled with cluster=${pulsar_cluster}. The value of ${pulsar_cluster} is the pulsar cluster name you configured in broker.conf.

Broker has the following kinds of metrics:

Namespace metrics​

Namespace metrics are only exposed when exposeTopicLevelMetricsInPrometheus is set to false.

All the namespace metrics are labelled with the following labels:

  • cluster: cluster=${pulsar_cluster}. ${pulsar_cluster} is the cluster name that you configured in broker.conf.
  • namespace: namespace=${pulsar_namespace}. ${pulsar_namespace} is the namespace name.
NameTypeDescription
pulsar_topics_countGaugeThe number of Pulsar topics of the namespace owned by this broker.
pulsar_subscriptions_countGaugeThe number of Pulsar subscriptions of the namespace served by this broker.
pulsar_producers_countGaugeThe number of active producers of the namespace connected to this broker.
pulsar_consumers_countGaugeThe number of active consumers of the namespace connected to this broker.
pulsar_rate_inGaugeThe total message rate of the namespace coming into this broker (messages/second).
pulsar_rate_outGaugeThe total message rate of the namespace going out from this broker (messages/second).
pulsar_throughput_inGaugeThe total throughput of the namespace coming into this broker (bytes/second).
pulsar_throughput_outGaugeThe total throughput of the namespace going out from this broker (bytes/second).
pulsar_storage_sizeGaugeThe total storage size of the topics in this namespace owned by this broker (bytes).
pulsar_storage_backlog_sizeGaugeThe total backlog size of the topics of this namespace owned by this broker (messages).
pulsar_storage_offloaded_sizeGaugeThe total amount of the data in this namespace offloaded to the tiered storage (bytes).
pulsar_storage_write_rateGaugeThe total message batches (entries) written to the storage for this namespace (message batches / second).
pulsar_storage_read_rateGaugeThe total message batches (entries) read from the storage for this namespace (message batches / second).
pulsar_subscription_delayedGaugeThe total message batches (entries) are delayed for dispatching.
pulsarstorage_write_latency_le*HistogramThe entry rate of a namespace that the storage write latency is smaller with a given threshold.
Available thresholds:
  • pulsar_storage_write_latency_le_0_5: <= 0.5ms
  • pulsar_storage_write_latency_le_1: <= 1ms
  • pulsar_storage_write_latency_le_5: <= 5ms
  • pulsar_storage_write_latency_le_10: <= 10ms
  • pulsar_storage_write_latency_le_20: <= 20ms
  • pulsar_storage_write_latency_le_50: <= 50ms
  • pulsar_storage_write_latency_le_100: <= 100ms
  • pulsar_storage_write_latency_le_200: <= 200ms
  • pulsar_storage_write_latency_le_1000: <= 1s
  • pulsar_storage_write_latency_le_overflow: > 1s
pulsarentry_size_le*HistogramThe entry rate of a namespace that the entry size is smaller with a given threshold.
Available thresholds:
  • pulsar_entry_size_le_128: <= 128 bytes
  • pulsar_entry_size_le_512: <= 512 bytes
  • pulsar_entry_size_le_1_kb: <= 1 KB
  • pulsar_entry_size_le_2_kb: <= 2 KB
  • pulsar_entry_size_le_4_kb: <= 4 KB
  • pulsar_entry_size_le_16_kb: <= 16 KB
  • pulsar_entry_size_le_100_kb: <= 100 KB
  • pulsar_entry_size_le_1_mb: <= 1 MB
  • pulsar_entry_size_le_overflow: > 1 MB

Replication metrics​

If a namespace is configured to be replicated between multiple Pulsar clusters, the corresponding replication metrics will also be exposed when replicationMetricsEnabled is enabled.

All the replication metrics will also be labelled with remoteCluster=${pulsar_remote_cluster}.

NameTypeDescription
pulsar_replication_rate_inGaugeThe total message rate of the namespace replicating from remote cluster (messages/second).
pulsar_replication_rate_outGaugeThe total message rate of the namespace replicating to remote cluster (messages/second).
pulsar_replication_throughput_inGaugeThe total throughput of the namespace replicating from remote cluster (bytes/second).
pulsar_replication_throughput_outGaugeThe total throughput of the namespace replicating to remote cluster (bytes/second).
pulsar_replication_backlogGaugeThe total backlog of the namespace replicating to remote cluster (messages).

Topic metrics​

Topic metrics are only exposed when exposeTopicLevelMetricsInPrometheus is set to true.

All the topic metrics are labelled with the following labels:

  • cluster: cluster=${pulsar_cluster}. ${pulsar_cluster} is the cluster name that you configured in broker.conf.
  • namespace: namespace=${pulsar_namespace}. ${pulsar_namespace} is the namespace name.
  • topic: topic=${pulsar_topic}. ${pulsar_topic} is the topic name.
NameTypeDescription
pulsar_subscriptions_countGaugeThe number of Pulsar subscriptions of the topic served by this broker.
pulsar_producers_countGaugeThe number of active producers of the topic connected to this broker.
pulsar_consumers_countGaugeThe number of active consumers of the topic connected to this broker.
pulsar_rate_inGaugeThe total message rate of the topic coming into this broker (messages/second).
pulsar_rate_outGaugeThe total message rate of the topic going out from this broker (messages/second).
pulsar_throughput_inGaugeThe total throughput of the topic coming into this broker (bytes/second).
pulsar_throughput_outGaugeThe total throughput of the topic going out from this broker (bytes/second).
pulsar_storage_sizeGaugeThe total storage size of the topics in this topic owned by this broker (bytes).
pulsar_storage_backlog_sizeGaugeThe total backlog size of the topics of this topic owned by this broker (messages).
pulsar_storage_offloaded_sizeGaugeThe total amount of the data in this topic offloaded to the tiered storage (bytes).
pulsar_storage_backlog_quota_limitGaugeThe total amount of the data in this topic that limit the backlog quota (bytes).
pulsar_storage_write_rateGaugeThe total message batches (entries) written to the storage for this topic (message batches / second).
pulsar_storage_read_rateGaugeThe total message batches (entries) read from the storage for this topic (message batches / second).
pulsar_subscription_delayedGaugeThe total message batches (entries) are delayed for dispatching.
pulsarstorage_write_latency_le*HistogramThe entry rate of a topic that the storage write latency is smaller with a given threshold.
Available thresholds:
  • pulsar_storage_write_latency_le_0_5: <= 0.5ms
  • pulsar_storage_write_latency_le_1: <= 1ms
  • pulsar_storage_write_latency_le_5: <= 5ms
  • pulsar_storage_write_latency_le_10: <= 10ms
  • pulsar_storage_write_latency_le_20: <= 20ms
  • pulsar_storage_write_latency_le_50: <= 50ms
  • pulsar_storage_write_latency_le_100: <= 100ms
  • pulsar_storage_write_latency_le_200: <= 200ms
  • pulsar_storage_write_latency_le_1000: <= 1s
  • pulsar_storage_write_latency_le_overflow: > 1s
pulsarentry_size_le*HistogramThe entry rate of a topic that the entry size is smaller with a given threshold.
Available thresholds:
  • pulsar_entry_size_le_128: <= 128 bytes
  • pulsar_entry_size_le_512: <= 512 bytes
  • pulsar_entry_size_le_1_kb: <= 1 KB
  • pulsar_entry_size_le_2_kb: <= 2 KB
  • pulsar_entry_size_le_4_kb: <= 4 KB
  • pulsar_entry_size_le_16_kb: <= 16 KB
  • pulsar_entry_size_le_100_kb: <= 100 KB
  • pulsar_entry_size_le_1_mb: <= 1 MB
  • pulsar_entry_size_le_overflow: > 1 MB
pulsar_in_bytes_totalCounterThe total number of bytes received for this topic
pulsar_in_messages_totalCounterThe total number of messages received for this topic
pulsar_out_bytes_totalCounterThe total number of bytes read from this topic
pulsar_out_messages_totalCounterThe total number of messages read from this topic

Replication metrics​

If a namespace that a topic belongs to is configured to be replicated between multiple Pulsar clusters, the corresponding replication metrics will also be exposed when replicationMetricsEnabled is enabled.

All the replication metrics will also be labelled with remoteCluster=${pulsar_remote_cluster}.

NameTypeDescription
pulsar_replication_rate_inGaugeThe total message rate of the topic replicating from remote cluster (messages/second).
pulsar_replication_rate_outGaugeThe total message rate of the topic replicating to remote cluster (messages/second).
pulsar_replication_throughput_inGaugeThe total throughput of the topic replicating from remote cluster (bytes/second).
pulsar_replication_throughput_outGaugeThe total throughput of the topic replicating to remote cluster (bytes/second).
pulsar_replication_backlogGaugeThe total backlog of the topic replicating to remote cluster (messages).

ManagedLedgerCache metrics​

All the ManagedLedgerCache metrics are labelled with the following labels:

  • cluster: cluster=${pulsar_cluster}. ${pulsar_cluster} is the cluster name that you configured in broker.conf.
NameTypeDescription
pulsar_ml_cache_evictionsGaugeThe number of cache evictions during the last minute.
pulsar_ml_cache_hits_rateGaugeThe number of cache hits per second.
pulsar_ml_cache_hits_throughputGaugeThe amount of data is retrieved from the cache in byte/s
pulsar_ml_cache_misses_rateGaugeThe number of cache misses per second
pulsar_ml_cache_misses_throughputGaugeThe amount of data is retrieved from the cache in byte/s
pulsar_ml_cache_pool_active_allocationsGaugeThe number of currently active allocations in direct arena
pulsar_ml_cache_pool_active_allocations_hugeGaugeThe number of currently active huge allocation in direct arena
pulsar_ml_cache_pool_active_allocations_normalGaugeThe number of currently active normal allocations in direct arena
pulsar_ml_cache_pool_active_allocations_smallGaugeThe number of currently active small allocations in direct arena
pulsar_ml_cache_pool_active_allocations_tinyGaugeThe number of currently active tiny allocations in direct arena
pulsar_ml_cache_pool_allocatedGaugeThe total allocated memory of chunk lists in direct arena
pulsar_ml_cache_pool_usedGaugeThe total used memory of chunk lists in direct arena
pulsar_ml_cache_used_sizeGaugeThe size in byte used to store the entries payloads
pulsar_ml_countGaugeThe number of currently opened managed ledgers

ManagedLedger metrics​

All the managedLedger metrics are labelled with the following labels:

  • cluster: cluster=${pulsar_cluster}. ${pulsar_cluster} is the cluster name that you configured in broker.conf.
  • namespace: namespace=${pulsar_namespace}. ${pulsar_namespace} is the namespace name.
  • quantile: quantile=${quantile}. Quantile is only for Histogram type metric, and represents the threshold for given Buckets.
NameTypeDescription
pulsar_ml_AddEntryBytesRateGaugeThe bytes/s rate of messages added
pulsar_ml_AddEntryErrorsGaugeThe number of addEntry requests that failed
pulsar_ml_AddEntryLatencyBucketsHistogramThe add entry latency of a ledger with a given quantile (threshold).
Available quantile:
  • quantile="0.0_0.5" is AddEntryLatency between (0.0ms, 0.5ms]
  • quantile="0.5_1.0" is AddEntryLatency between (0.5ms, 1.0ms]
  • quantile="1.0_5.0" is AddEntryLatency between (1ms, 5ms]
  • quantile="5.0_10.0" is AddEntryLatency between (5ms, 10ms]
  • quantile="10.0_20.0" is AddEntryLatency between (10ms, 20ms]
  • quantile="20.0_50.0" is AddEntryLatency between (20ms, 50ms]
  • quantile="50.0_100.0" is AddEntryLatency between (50ms, 100ms]
  • quantile="100.0_200.0" is AddEntryLatency between (100ms, 200ms]
  • quantile="200.0_1000.0" is AddEntryLatency between (200ms, 1s]
pulsar_ml_AddEntryLatencyBuckets_OVERFLOWGaugeThe add entry latency > 1s
pulsar_ml_AddEntryMessagesRateGaugeThe msg/s rate of messages added
pulsar_ml_AddEntrySucceedGaugeThe number of addEntry requests that succeeded
pulsar_ml_EntrySizeBucketsHistogramThe add entry size of a ledger with given quantile.
Available quantile:
  • quantile="0.0_128.0" is EntrySize between (0byte, 128byte]
  • quantile="128.0_512.0" is EntrySize between (128byte, 512byte]
  • quantile="512.0_1024.0" is EntrySize between (512byte, 1KB]
  • quantile="1024.0_2048.0" is EntrySize between (1KB, 2KB]
  • quantile="2048.0_4096.0" is EntrySize between (2KB, 4KB]
  • quantile="4096.0_16384.0" is EntrySize between (4KB, 16KB]
  • quantile="16384.0_102400.0" is EntrySize between (16KB, 100KB]
  • quantile="102400.0_1232896.0" is EntrySize between (100KB, 1MB]
pulsar_ml_EntrySizeBuckets_OVERFLOWGaugeThe add entry size > 1MB
pulsar_ml_LedgerSwitchLatencyBucketsHistogramThe ledger switch latency with given quantile.
Available quantile:
  • quantile="0.0_0.5" is EntrySize between (0ms, 0.5ms]
  • quantile="0.5_1.0" is EntrySize between (0.5ms, 1ms]
  • quantile="1.0_5.0" is EntrySize between (1ms, 5ms]
  • quantile="5.0_10.0" is EntrySize between (5ms, 10ms]
  • quantile="10.0_20.0" is EntrySize between (10ms, 20ms]
  • quantile="20.0_50.0" is EntrySize between (20ms, 50ms]
  • quantile="50.0_100.0" is EntrySize between (50ms, 100ms]
  • quantile="100.0_200.0" is EntrySize between (100ms, 200ms]
  • quantile="200.0_1000.0" is EntrySize between (200ms, 1000ms]
pulsar_ml_LedgerSwitchLatencyBuckets_OVERFLOWGaugeThe ledger switch latency > 1s
pulsar_ml_MarkDeleteRateGaugeThe rate of mark-delete ops/s
pulsar_ml_NumberOfMessagesInBacklogGaugeThe number of backlog messages for all the consumers
pulsar_ml_ReadEntriesBytesRateGaugeThe bytes/s rate of messages read
pulsar_ml_ReadEntriesErrorsGaugeThe number of readEntries requests that failed
pulsar_ml_ReadEntriesRateGaugeThe msg/s rate of messages read
pulsar_ml_ReadEntriesSucceededGaugeThe number of readEntries requests that succeeded
pulsar_ml_StoredMessagesSizeGaugeThe total size of the messages in active ledgers (accounting for the multiple copies stored)

LoadBalancing metrics​

All the loadbalancing metrics are labelled with the following labels:

  • cluster: cluster=${pulsar_cluster}. ${pulsar_cluster} is the cluster name that you configured in broker.conf.
  • broker: broker=${broker}. ${broker} is the ip address of the broker
  • metric: metric="loadBalancing".
NameTypeDescription
pulsar_lb_bandwidth_in_usageGaugeThe broker inbound bandwith usage (in percent).
pulsar_lb_bandwidth_out_usageGaugeThe broker outbound bandwith usage (in percent).
pulsar_lb_cpu_usageGaugeThe broker cpu usage (in percent).
pulsar_lb_directMemory_usageGaugeThe broker process direct memory usage (in percent).
pulsar_lb_memory_usageGaugeThe broker process memory usage (in percent).

BundleUnloading metrics​

All the bundleUnloading metrics are labelled with the following labels:

  • cluster: cluster=${pulsar_cluster}. ${pulsar_cluster} is the cluster name that you configured in broker.conf.
  • metric: metric="bundleUnloading".
NameTypeDescription
pulsar_lb_unload_broker_countCounterUnload broker count in this bundle unloading
pulsar_lb_unload_bundle_countCounterBundle unload count in this bundle unloading

BundleSplit metrics​

All the bundleUnloading metrics are labelled with the following labels:

  • cluster: cluster=${pulsar_cluster}. ${pulsar_cluster} is the cluster name that you configured in broker.conf.
  • metric: metric="bundlesSplit".
NameTypeDescription
pulsar_lb_bundles_split_countCounterThe total count of bundle split in this leader broker

Subscription metrics​

Subscription metrics are only exposed when exposeTopicLevelMetricsInPrometheus is set to true.

All the subscription metrics are labelled with the following labels:

  • cluster: cluster=${pulsar_cluster}. ${pulsar_cluster} is the cluster name that you configured in broker.conf.
  • namespace: namespace=${pulsar_namespace}. ${pulsar_namespace} is the namespace name.
  • topic: topic=${pulsar_topic}. ${pulsar_topic} is the topic name.
  • subscription: subscription=${subscription}. ${subscription} is the topic subscription name.
NameTypeDescription
pulsar_subscription_back_logGaugeThe total backlog of a subscription (messages).
pulsar_subscription_delayedGaugeThe total number of messages are delayed to be dispatched for a subscription (messages).
pulsar_subscription_msg_rate_redeliverGaugeThe total message rate for message being redelivered (messages/second).
pulsar_subscription_unacked_messagesGaugeThe total number of unacknowledged messages of a subscription (messages).
pulsar_subscription_blocked_on_unacked_messagesGaugeIndicate whether a subscription is blocked on unacknowledged messages or not.
  • 1 means the subscription is blocked on waiting unacknowledged messages to be acked.
  • 0 means the subscription is not blocked on waiting unacknowledged messages to be acked.
pulsar_subscription_msg_rate_outGaugeThe total message dispatch rate for a subscription (messages/second).
pulsar_subscription_msg_throughput_outGaugeThe total message dispatch throughput for a subscription (bytes/second).

Consumer metrics​

Consumer metrics are only exposed when both exposeTopicLevelMetricsInPrometheus and exposeConsumerLevelMetricsInPrometheus are set to true.

All the consumer metrics are labelled with the following labels:

  • cluster: cluster=${pulsar_cluster}. ${pulsar_cluster} is the cluster name that you configured in broker.conf.
  • namespace: namespace=${pulsar_namespace}. ${pulsar_namespace} is the namespace name.
  • topic: topic=${pulsar_topic}. ${pulsar_topic} is the topic name.
  • subscription: subscription=${subscription}. ${subscription} is the topic subscription name.
  • consumer_name: consumer_name=${consumer_name}. ${consumer_name} is the topic consumer name.
  • consumer_id: consumer_id=${consumer_id}. ${consumer_id} is the topic consumer id.
NameTypeDescription
pulsar_consumer_msg_rate_redeliverGaugeThe total message rate for message being redelivered (messages/second).
pulsar_consumer_unacked_messagesGaugeThe total number of unacknowledged messages of a consumer (messages).
pulsar_consumer_blocked_on_unacked_messagesGaugeIndicate whether a consumer is blocked on unacknowledged messages or not.
  • 1 means the consumer is blocked on waiting unacknowledged messages to be acked.
  • 0 means the consumer is not blocked on waiting unacknowledged messages to be acked.
pulsar_consumer_msg_rate_outGaugeThe total message dispatch rate for a consumer (messages/second).
pulsar_consumer_msg_throughput_outGaugeThe total message dispatch throughput for a consumer (bytes/second).
pulsar_consumer_available_permitsGaugeThe available permits for for a consumer.

Managed ledger bookie client metrics​

All the managed ledger bookie client metrics labelled with the following labels:

  • cluster: cluster=${pulsar_cluster}. ${pulsar_cluster} is the cluster name that you configured in broker.conf.
NameTypeDescription
pulsarmanagedLedger_client_bookkeeper_ml_scheduler_completed_tasks*GaugeThe number of tasks the scheduler executor execute completed.
The number of metrics determined by the scheduler executor thread number configured by managedLedgerNumSchedulerThreads in broker.conf.
pulsarmanagedLedger_client_bookkeeper_ml_scheduler_queue*GaugeThe number of tasks queued in the scheduler executor's queue.
The number of metrics determined by scheduler executor's thread number configured by managedLedgerNumSchedulerThreads in broker.conf.
pulsarmanagedLedger_client_bookkeeper_ml_scheduler_total_tasks*GaugeThe total number of tasks the scheduler executor received.
The number of metrics determined by scheduler executor's thread number configured by managedLedgerNumSchedulerThreads in broker.conf.
pulsarmanagedLedger_client_bookkeeper_ml_workers_completed_tasks*GaugeThe number of tasks the worker executor execute completed.
The number of metrics determined by the number of worker task thread number configured by managedLedgerNumWorkerThreads in broker.conf
pulsarmanagedLedger_client_bookkeeper_ml_workers_queue*GaugeThe number of tasks queued in the worker executor's queue.
The number of metrics determined by scheduler executor's thread number configured by managedLedgerNumWorkerThreads in broker.conf.
pulsarmanagedLedger_client_bookkeeper_ml_workers_total_tasks*GaugeThe total number of tasks the worker executor received.
The number of metrics determined by worker executor's thread number configured by managedLedgerNumWorkerThreads in broker.conf.
pulsar_managedLedger_client_bookkeeper_ml_scheduler_task_executionSummaryThe scheduler task execution latency calculated in milliseconds.
pulsar_managedLedger_client_bookkeeper_ml_scheduler_task_queuedSummaryThe scheduler task queued latency calculated in milliseconds.
pulsar_managedLedger_client_bookkeeper_ml_workers_task_executionSummaryThe worker task execution latency calculated in milliseconds.
pulsar_managedLedger_client_bookkeeper_ml_workers_task_queuedSummaryThe worker task queued latency calculated in milliseconds.

Jetty metrics​

For a functions-worker running separately from brokers, its Jetty metrics are only exposed when includeStandardPrometheusMetrics is set to true.

All the jetty metrics are labelled with the following labels:

  • cluster: cluster=${pulsar_cluster}. ${pulsar_cluster} is the cluster name that you have configured in the broker.conf file.
NameTypeDescription
jetty_requests_totalCounterNumber of requests.
jetty_requests_activeGaugeNumber of requests currently active.
jetty_requests_active_maxGaugeMaximum number of requests that have been active at once.
jetty_request_time_max_secondsGaugeMaximum time spent handling requests.
jetty_request_time_seconds_totalCounterTotal time spent in all request handling.
jetty_dispatched_totalCounterNumber of dispatches.
jetty_dispatched_activeGaugeNumber of dispatches currently active.
jetty_dispatched_active_maxGaugeMaximum number of active dispatches being handled.
jetty_dispatched_time_maxGaugeMaximum time spent in dispatch handling.
jetty_dispatched_time_seconds_totalCounterTotal time spent in dispatch handling.
jetty_async_requests_totalCounterTotal number of async requests.
jetty_async_requests_waitingGaugeCurrently waiting async requests.
jetty_async_requests_waiting_maxGaugeMaximum number of waiting async requests.
jetty_async_dispatches_totalCounterNumber of requested that have been asynchronously dispatched.
jetty_expires_totalCounterNumber of async requests requests that have expired.
jetty_responses_totalCounterNumber of responses, labeled by status code. The code label can be "1xx", "2xx", "3xx", "4xx", or "5xx".
jetty_stats_secondsGaugeTime in seconds stats have been collected for.
jetty_responses_bytes_totalCounterTotal number of bytes across all responses.

Pulsar Functions

All the Pulsar Functions metrics are labelled with the following labels:

  • cluster: cluster=${pulsar_cluster}. ${pulsar_cluster} is the cluster name that you configured in broker.conf.
  • namespace: namespace=${pulsar_namespace}. ${pulsar_namespace} is the namespace name.
NameTypeDescription
pulsar_function_processed_successfully_totalCounterTotal number of messages processed successfully.
pulsar_function_processed_successfully_total_1minCounterTotal number of messages processed successfully in the last 1 minute.
pulsar_function_system_exceptions_totalCounterTotal number of system exceptions.
pulsar_function_system_exceptions_total_1minCounterTotal number of system exceptions in the last 1 minute.
pulsar_function_user_exceptions_totalCounterTotal number of user exceptions.
pulsar_function_user_exceptions_total_1minCounterTotal number of user exceptions in the last 1 minute.
pulsar_function_process_latency_msSummaryProcess latency in milliseconds.
pulsar_function_process_latency_ms_1minSummaryProcess latency in milliseconds in the last 1 minute.
pulsar_function_last_invocationGaugeThe timestamp of the last invocation of the function.
pulsar_function_received_totalCounterTotal number of messages received from source.
pulsar_function_received_total_1minCounterTotal number of messages received from source in the last 1 minute.

Proxy

All the proxy metrics are labelled with the following labels:

  • cluster: cluster=${pulsar_cluster}. ${pulsar_cluster} is the cluster name that you configured in broker.conf.
  • kubernetes_pod_name: kubernetes_pod_name=${kubernetes_pod_name}. ${kubernetes_pod_name} is the Kubernetes pod name.
NameTypeDescription
pulsar_proxy_active_connectionsGaugeNumber of connections currently active in the proxy.
pulsar_proxy_new_connectionsCounterCounter of connections being opened in the proxy.
pulsar_proxy_rejected_connectionsCounterCounter for connections rejected due to throttling.
pulsar_proxy_binary_opsCounterCounter of proxy operations.
pulsar_proxy_binary_bytesCounterCounter of proxy bytes.

Pulsar SQL Worker

NameTypeDescription
split_bytes_readCounterNumber of bytes read from BookKeeper.
split_num_messages_deserializedCounterNumber of messages deserialized.
split_num_record_deserializedCounterNumber of records deserialized.
split_bytes_read_per_querySummaryTotal number of bytes read per query.
split_entry_deserialize_timeSummaryTime spent on derserializing entries.
split_entry_deserialize_time_per_querySummaryTime spent on derserializing entries per query.
split_entry_queue_dequeue_wait_timeSummaryTime spend on waiting to get entry from entry queue because it is empty.
split_entry_queue_dequeue_wait_time_per_querySummaryTotal time spent on waiting to get entry from entry queue per query.
split_message_queue_dequeue_wait_time_per_querySummaryTime spent on waiting to dequeue from message queue because is is empty per query.
split_message_queue_enqueue_wait_timeSummaryTime spent on waiting for message queue enqueue because the message queue is full.
split_message_queue_enqueue_wait_time_per_querySummaryTime spent on waiting for message queue enqueue because the message queue is full per query.
split_num_entries_per_batchSummaryNumber of entries per batch.
split_num_entries_per_querySummaryNumber of entries per query.
split_num_messages_deserialized_per_entrySummaryNumber of messages deserialized per entry.
split_num_messages_deserialized_per_querySummaryNumber of messages deserialized per query.
split_read_attemptsSummaryNumber of read attempts (fail if queues are full).
split_read_attempts_per_querySummaryNumber of read attempts per query.
split_read_latency_per_batchSummaryLatency of reads per batch.
split_read_latency_per_querySummaryTotal read latency per query.
split_record_deserialize_timeSummaryTime spent on deserializing message to record. For example, Avro, JSON, and so on.
split_record_deserialize_time_per_querySummaryTime spent on deserializing message to record per query.
split_total_execution_timeSummaryTotal execution time .

Monitor​

You can set up a Prometheus instance to collect all the metrics exposed at Pulsar components and set up Grafana dashboards to display the metrics and monitor your Pulsar cluster.

The following are some Grafana dashboards examples:

  • pulsar-grafana: A grafana dashboard that displays metrics collected in Prometheus for Pulsar clusters running on Kubernetes.
  • apache-pulsar-grafana-dashboard: A collection of grafana dashboard templates for different Pulsar components running on both Kubernetes and on-premise machines.