Skip to content

Commit

Permalink
metrics: stabilize RuntimeMetrics::worker_count
Browse files Browse the repository at this point in the history
This PR stabilizes a single metric API to start the process of stabilizing metrics.
Future work will continue to stabilize more metrics.

Refs: tokio-rs#6546
  • Loading branch information
rcoh committed May 16, 2024
1 parent d221c50 commit 78890bb
Show file tree
Hide file tree
Showing 18 changed files with 126 additions and 65 deletions.
37 changes: 34 additions & 3 deletions tokio/src/macros/cfg.rs
Original file line number Diff line number Diff line change
Expand Up @@ -215,22 +215,41 @@ macro_rules! cfg_macros {
}
}

// When running tests in loom, mocks are used instead of metrics.
macro_rules! cfg_metrics {
($($item:item)*) => {
$(
// For now, metrics is only disabled in loom tests.
// When stabilized, it might have a dedicated feature flag.
#[cfg(not(loom))]
$item
)*
}
}

// See https://github.com/tokio-rs/tokio/issues/6546, https://github.com/tokio-rs/tokio/issues/4073
macro_rules! cfg_unstable_metrics {
($($item:item)*) => {
$(
#[cfg(all(tokio_unstable, not(loom)))]
#[cfg_attr(docsrs, doc(cfg(tokio_unstable)))]
$item
)*
}
}

macro_rules! cfg_not_unstable_metrics {
($($item:item)*) => {
$(
#[cfg(any(not(tokio_unstable), loom))]
$item
)*
}
}

// Swap in mock metrics when loom is active
macro_rules! cfg_not_metrics {
($($item:item)*) => {
$(
#[cfg(not(all(tokio_unstable, not(loom))))]
#[cfg(loom)]
$item
)*
}
Expand Down Expand Up @@ -487,6 +506,18 @@ macro_rules! cfg_unstable {
};
}

macro_rules! cfg_net_and_unstable {
($($item:item)*) => {
$(
#[cfg(tokio_unstable)]
#[cfg_attr(docsrs, doc(cfg(tokio_unstable)))]
#[cfg(feature = "net")]
#[cfg_attr(docsrs, doc(cfg(feature = "net")))]
$item
)*
};
}

macro_rules! cfg_not_trace {
($($item:item)*) => {
$(
Expand Down
4 changes: 2 additions & 2 deletions tokio/src/runtime/blocking/pool.rs
Original file line number Diff line number Diff line change
Expand Up @@ -40,7 +40,7 @@ impl SpawnerMetrics {
self.num_idle_threads.load(Ordering::Relaxed)
}

cfg_metrics! {
cfg_unstable_metrics! {
fn queue_depth(&self) -> usize {
self.queue_depth.load(Ordering::Relaxed)
}
Expand Down Expand Up @@ -474,7 +474,7 @@ impl Spawner {
}
}

cfg_metrics! {
cfg_unstable_metrics! {
impl Spawner {
pub(crate) fn num_threads(&self) -> usize {
self.inner.metrics.num_threads()
Expand Down
2 changes: 1 addition & 1 deletion tokio/src/runtime/builder.rs
Original file line number Diff line number Diff line change
Expand Up @@ -956,7 +956,7 @@ impl Builder {
}
}

cfg_metrics! {
cfg_unstable_metrics! {
/// Enables tracking the distribution of task poll times.
///
/// Task poll times are not instrumented by default as doing so requires
Expand Down
4 changes: 2 additions & 2 deletions tokio/src/runtime/coop.rs
Original file line number Diff line number Diff line change
Expand Up @@ -197,7 +197,7 @@ cfg_coop! {
}

cfg_rt! {
cfg_metrics! {
cfg_unstable_metrics! {
#[inline(always)]
fn inc_budget_forced_yield_count() {
let _ = context::with_current(|handle| {
Expand All @@ -206,7 +206,7 @@ cfg_coop! {
}
}

cfg_not_metrics! {
cfg_not_unstable_metrics! {
#[inline(always)]
fn inc_budget_forced_yield_count() {}
}
Expand Down
11 changes: 11 additions & 0 deletions tokio/src/runtime/handle.rs
Original file line number Diff line number Diff line change
Expand Up @@ -401,6 +401,17 @@ cfg_metrics! {
impl Handle {
/// Returns a view that lets you get information about how the runtime
/// is performing.
///
/// # Examples
/// ```
/// use tokio::runtime::Handle;
///
/// #[tokio::main]
/// async fn main() {
/// let metrics = Handle::current().metrics();
/// println!("The runtime has {} workers", metrics.num_workers());
/// }
/// ```
pub fn metrics(&self) -> RuntimeMetrics {
RuntimeMetrics::new(self.clone())
}
Expand Down
2 changes: 1 addition & 1 deletion tokio/src/runtime/io/metrics.rs
Original file line number Diff line number Diff line change
Expand Up @@ -17,7 +17,7 @@ cfg_not_rt_and_metrics_and_net! {

cfg_net! {
cfg_rt! {
cfg_metrics! {
cfg_unstable_metrics! {
pub(crate) use crate::runtime::IoDriverMetrics;
}
}
Expand Down
40 changes: 24 additions & 16 deletions tokio/src/runtime/metrics/mod.rs
Original file line number Diff line number Diff line change
Expand Up @@ -9,27 +9,35 @@
#![allow(clippy::module_inception)]

cfg_metrics! {
mod batch;
pub(crate) use batch::MetricsBatch;

mod histogram;
pub(crate) use histogram::{Histogram, HistogramBatch, HistogramBuilder};
#[allow(unreachable_pub)] // rust-lang/rust#57411
pub use histogram::HistogramScale;

mod runtime;
#[allow(unreachable_pub)] // rust-lang/rust#57411
pub use runtime::RuntimeMetrics;

mod scheduler;
pub(crate) use scheduler::SchedulerMetrics;
cfg_unstable_metrics! {
mod batch;
pub(crate) use batch::MetricsBatch;

mod histogram;
pub(crate) use histogram::{HistogramBatch, HistogramBuilder, Histogram};

mod scheduler;
pub(crate) use scheduler::SchedulerMetrics;

mod worker;
pub(crate) use worker::WorkerMetrics;

cfg_net_and_unstable! {
mod io;
pub(crate) use io::IoDriverMetrics;
}

#[allow(unreachable_pub)] // rust-lang/rust#57411
pub use histogram::HistogramScale;
}

mod worker;
pub(crate) use worker::WorkerMetrics;
cfg_not_unstable_metrics! {
mod mock;

cfg_net! {
mod io;
pub(crate) use io::IoDriverMetrics;
pub(crate) use mock::{SchedulerMetrics, WorkerMetrics, MetricsBatch, HistogramBuilder};
}
}

Expand Down
16 changes: 12 additions & 4 deletions tokio/src/runtime/metrics/runtime.rs
Original file line number Diff line number Diff line change
@@ -1,9 +1,5 @@
use crate::runtime::Handle;

use std::ops::Range;
use std::sync::atomic::Ordering::Relaxed;
use std::time::Duration;

/// Handle to the runtime's metrics.
///
/// This handle is internally reference-counted and can be freely cloned. A
Expand All @@ -15,6 +11,12 @@ pub struct RuntimeMetrics {
handle: Handle,
}

cfg_unstable_metrics! {
use std::time::Duration;
use std::sync::atomic::Ordering::Relaxed;
use std::ops::Range;
}

impl RuntimeMetrics {
pub(crate) fn new(handle: Handle) -> RuntimeMetrics {
RuntimeMetrics { handle }
Expand Down Expand Up @@ -43,6 +45,8 @@ impl RuntimeMetrics {
self.handle.inner.num_workers()
}

cfg_unstable_metrics! {

/// Returns the number of additional threads spawned by the runtime.
///
/// The number of workers is set by configuring `max_blocking_threads` on
Expand Down Expand Up @@ -833,10 +837,13 @@ impl RuntimeMetrics {
pub fn blocking_queue_depth(&self) -> usize {
self.handle.inner.blocking_queue_depth()
}
}
}

cfg_net! {
impl RuntimeMetrics {
cfg_unstable_metrics! {

/// Returns the number of file descriptors that have been registered with the
/// runtime's I/O driver.
///
Expand Down Expand Up @@ -921,4 +928,5 @@ cfg_net! {
.unwrap_or(0)
}
}
}
}
18 changes: 9 additions & 9 deletions tokio/src/runtime/mod.rs
Original file line number Diff line number Diff line change
Expand Up @@ -388,21 +388,21 @@ cfg_rt! {
mod thread_id;
pub(crate) use thread_id::ThreadId;

pub(crate) mod metrics;
cfg_metrics! {
mod metrics;
pub use metrics::{RuntimeMetrics, HistogramScale};
pub use metrics::RuntimeMetrics;

pub(crate) use metrics::{MetricsBatch, SchedulerMetrics, WorkerMetrics, HistogramBuilder};
cfg_unstable_metrics! {
pub use metrics::HistogramScale;

cfg_net! {
pub(crate) use metrics::IoDriverMetrics;
cfg_net! {
pub(crate) use metrics::IoDriverMetrics;
}
}
}

cfg_not_metrics! {
pub(crate) mod metrics;
pub(crate) use metrics::{SchedulerMetrics, WorkerMetrics, MetricsBatch, HistogramBuilder};
}
pub(crate) use metrics::{MetricsBatch, SchedulerMetrics, WorkerMetrics, HistogramBuilder};


/// After thread starts / before thread stops
type Callback = std::sync::Arc<dyn Fn() + Send + Sync>;
Expand Down
18 changes: 8 additions & 10 deletions tokio/src/runtime/runtime.rs
Original file line number Diff line number Diff line change
Expand Up @@ -455,6 +455,14 @@ impl Runtime {
pub fn shutdown_background(self) {
self.shutdown_timeout(Duration::from_nanos(0));
}

cfg_metrics! {
/// Returns a view that lets you get information about how the runtime
/// is performing.
pub fn metrics(&self) -> crate::runtime::RuntimeMetrics {
self.handle.metrics()
}
}
}

#[allow(clippy::single_match)] // there are comments in the error branch, so we don't want if-let
Expand Down Expand Up @@ -486,13 +494,3 @@ impl Drop for Runtime {
impl std::panic::UnwindSafe for Runtime {}

impl std::panic::RefUnwindSafe for Runtime {}

cfg_metrics! {
impl Runtime {
/// Returns a view that lets you get information about how the runtime
/// is performing.
pub fn metrics(&self) -> crate::runtime::RuntimeMetrics {
self.handle.metrics()
}
}
}
2 changes: 1 addition & 1 deletion tokio/src/runtime/scheduler/current_thread/mod.rs
Original file line number Diff line number Diff line change
Expand Up @@ -502,7 +502,7 @@ impl Handle {
}
}

cfg_metrics! {
cfg_unstable_metrics! {
impl Handle {
pub(crate) fn scheduler_metrics(&self) -> &SchedulerMetrics {
&self.shared.scheduler_metrics
Expand Down
2 changes: 1 addition & 1 deletion tokio/src/runtime/scheduler/inject.rs
Original file line number Diff line number Diff line change
Expand Up @@ -16,7 +16,7 @@ cfg_rt_multi_thread! {
mod rt_multi_thread;
}

cfg_metrics! {
cfg_unstable_metrics! {
mod metrics;
}

Expand Down
9 changes: 7 additions & 2 deletions tokio/src/runtime/scheduler/mod.rs
Original file line number Diff line number Diff line change
Expand Up @@ -164,8 +164,6 @@ cfg_rt! {
}

cfg_metrics! {
use crate::runtime::{SchedulerMetrics, WorkerMetrics};

impl Handle {
pub(crate) fn num_workers(&self) -> usize {
match self {
Expand All @@ -176,6 +174,13 @@ cfg_rt! {
Handle::MultiThreadAlt(handle) => handle.num_workers(),
}
}
}
}

cfg_unstable_metrics! {
use crate::runtime::{SchedulerMetrics, WorkerMetrics};
impl Handle {


pub(crate) fn num_blocking_threads(&self) -> usize {
match_flavor!(self, Handle(handle) => handle.num_blocking_threads())
Expand Down
4 changes: 1 addition & 3 deletions tokio/src/runtime/scheduler/multi_thread/handle.rs
Original file line number Diff line number Diff line change
Expand Up @@ -9,9 +9,7 @@ use crate::util::RngSeedGenerator;

use std::fmt;

cfg_metrics! {
mod metrics;
}
mod metrics;

cfg_taskdump! {
mod taskdump;
Expand Down
10 changes: 6 additions & 4 deletions tokio/src/runtime/scheduler/multi_thread/handle/metrics.rs
Original file line number Diff line number Diff line change
@@ -1,12 +1,13 @@
use super::Handle;

use crate::runtime::{SchedulerMetrics, WorkerMetrics};

impl Handle {
pub(crate) fn num_workers(&self) -> usize {
self.shared.worker_metrics.len()
}
}

cfg_unstable_metrics! {
impl Handle {
pub(crate) fn num_blocking_threads(&self) -> usize {
self.blocking_spawner.num_threads()
}
Expand All @@ -19,11 +20,11 @@ impl Handle {
self.shared.owned.active_tasks_count()
}

pub(crate) fn scheduler_metrics(&self) -> &SchedulerMetrics {
pub(crate) fn scheduler_metrics(&self) -> &crate::runtime::SchedulerMetrics {
&self.shared.scheduler_metrics
}

pub(crate) fn worker_metrics(&self, worker: usize) -> &WorkerMetrics {
pub(crate) fn worker_metrics(&self, worker: usize) -> &crate::runtime::WorkerMetrics {
&self.shared.worker_metrics[worker]
}

Expand All @@ -39,3 +40,4 @@ impl Handle {
self.blocking_spawner.queue_depth()
}
}
}
2 changes: 1 addition & 1 deletion tokio/src/runtime/scheduler/multi_thread/queue.rs
Original file line number Diff line number Diff line change
Expand Up @@ -546,7 +546,7 @@ impl<T> Steal<T> {
}
}

cfg_metrics! {
cfg_unstable_metrics! {
impl<T> Steal<T> {
pub(crate) fn len(&self) -> usize {
self.0.len() as _
Expand Down

0 comments on commit 78890bb

Please sign in to comment.