mirror of
https://github.com/rust-lang/rust-analyzer.git
synced 2025-09-26 11:59:49 +00:00
Wrap platform-specific QoS in r-a-specific “thread intent”
This commit is contained in:
parent
d0b001eed2
commit
74bc2a47e0
13 changed files with 354 additions and 295 deletions
|
@ -90,7 +90,7 @@ impl FlycheckHandle {
|
||||||
) -> FlycheckHandle {
|
) -> FlycheckHandle {
|
||||||
let actor = FlycheckActor::new(id, sender, config, workspace_root);
|
let actor = FlycheckActor::new(id, sender, config, workspace_root);
|
||||||
let (sender, receiver) = unbounded::<StateChange>();
|
let (sender, receiver) = unbounded::<StateChange>();
|
||||||
let thread = stdx::thread::Builder::new(stdx::thread::QoSClass::Utility)
|
let thread = stdx::thread::Builder::new(stdx::thread::ThreadIntent::Worker)
|
||||||
.name("Flycheck".to_owned())
|
.name("Flycheck".to_owned())
|
||||||
.spawn(move || actor.run(receiver))
|
.spawn(move || actor.run(receiver))
|
||||||
.expect("failed to spawn thread");
|
.expect("failed to spawn thread");
|
||||||
|
@ -409,7 +409,7 @@ impl CargoHandle {
|
||||||
|
|
||||||
let (sender, receiver) = unbounded();
|
let (sender, receiver) = unbounded();
|
||||||
let actor = CargoActor::new(sender, stdout, stderr);
|
let actor = CargoActor::new(sender, stdout, stderr);
|
||||||
let thread = stdx::thread::Builder::new(stdx::thread::QoSClass::Utility)
|
let thread = stdx::thread::Builder::new(stdx::thread::ThreadIntent::Worker)
|
||||||
.name("CargoHandle".to_owned())
|
.name("CargoHandle".to_owned())
|
||||||
.spawn(move || actor.run())
|
.spawn(move || actor.run())
|
||||||
.expect("failed to spawn thread");
|
.expect("failed to spawn thread");
|
||||||
|
|
|
@ -81,7 +81,7 @@ pub(crate) fn parallel_prime_caches(
|
||||||
let worker = prime_caches_worker.clone();
|
let worker = prime_caches_worker.clone();
|
||||||
let db = db.snapshot();
|
let db = db.snapshot();
|
||||||
|
|
||||||
stdx::thread::Builder::new(stdx::thread::QoSClass::Utility)
|
stdx::thread::Builder::new(stdx::thread::ThreadIntent::Worker)
|
||||||
.allow_leak(true)
|
.allow_leak(true)
|
||||||
.spawn(move || Cancelled::catch(|| worker(db)))
|
.spawn(move || Cancelled::catch(|| worker(db)))
|
||||||
.expect("failed to spawn thread");
|
.expect("failed to spawn thread");
|
||||||
|
|
|
@ -79,13 +79,15 @@ fn try_main(flags: flags::RustAnalyzer) -> Result<()> {
|
||||||
return Ok(());
|
return Ok(());
|
||||||
}
|
}
|
||||||
|
|
||||||
// rust-analyzer’s “main thread” is actually a secondary thread
|
// rust-analyzer’s “main thread” is actually
|
||||||
// with an increased stack size at the User Initiated QoS class.
|
// a secondary latency-sensitive thread with an increased stack size.
|
||||||
// We use this QoS class because any delay in the main loop
|
// We use this thread intent because any delay in the main loop
|
||||||
// will make actions like hitting enter in the editor slow.
|
// will make actions like hitting enter in the editor slow.
|
||||||
// rust-analyzer does not block the editor’s render loop,
|
with_extra_thread(
|
||||||
// so we don’t use User Interactive.
|
"LspServer",
|
||||||
with_extra_thread("LspServer", stdx::thread::QoSClass::UserInitiated, run_server)?;
|
stdx::thread::ThreadIntent::LatencySensitive,
|
||||||
|
run_server,
|
||||||
|
)?;
|
||||||
}
|
}
|
||||||
flags::RustAnalyzerCmd::Parse(cmd) => cmd.run()?,
|
flags::RustAnalyzerCmd::Parse(cmd) => cmd.run()?,
|
||||||
flags::RustAnalyzerCmd::Symbols(cmd) => cmd.run()?,
|
flags::RustAnalyzerCmd::Symbols(cmd) => cmd.run()?,
|
||||||
|
@ -143,10 +145,10 @@ const STACK_SIZE: usize = 1024 * 1024 * 8;
|
||||||
/// space.
|
/// space.
|
||||||
fn with_extra_thread(
|
fn with_extra_thread(
|
||||||
thread_name: impl Into<String>,
|
thread_name: impl Into<String>,
|
||||||
qos_class: stdx::thread::QoSClass,
|
thread_intent: stdx::thread::ThreadIntent,
|
||||||
f: impl FnOnce() -> Result<()> + Send + 'static,
|
f: impl FnOnce() -> Result<()> + Send + 'static,
|
||||||
) -> Result<()> {
|
) -> Result<()> {
|
||||||
let handle = stdx::thread::Builder::new(qos_class)
|
let handle = stdx::thread::Builder::new(thread_intent)
|
||||||
.name(thread_name.into())
|
.name(thread_name.into())
|
||||||
.stack_size(STACK_SIZE)
|
.stack_size(STACK_SIZE)
|
||||||
.spawn(f)?;
|
.spawn(f)?;
|
||||||
|
|
|
@ -4,7 +4,7 @@ use std::{fmt, panic, thread};
|
||||||
use ide::Cancelled;
|
use ide::Cancelled;
|
||||||
use lsp_server::ExtractError;
|
use lsp_server::ExtractError;
|
||||||
use serde::{de::DeserializeOwned, Serialize};
|
use serde::{de::DeserializeOwned, Serialize};
|
||||||
use stdx::thread::QoSClass;
|
use stdx::thread::ThreadIntent;
|
||||||
|
|
||||||
use crate::{
|
use crate::{
|
||||||
global_state::{GlobalState, GlobalStateSnapshot},
|
global_state::{GlobalState, GlobalStateSnapshot},
|
||||||
|
@ -104,7 +104,7 @@ impl<'a> RequestDispatcher<'a> {
|
||||||
None => return self,
|
None => return self,
|
||||||
};
|
};
|
||||||
|
|
||||||
self.global_state.task_pool.handle.spawn(QoSClass::Utility, {
|
self.global_state.task_pool.handle.spawn(ThreadIntent::Worker, {
|
||||||
let world = self.global_state.snapshot();
|
let world = self.global_state.snapshot();
|
||||||
move || {
|
move || {
|
||||||
let result = panic::catch_unwind(move || {
|
let result = panic::catch_unwind(move || {
|
||||||
|
@ -135,7 +135,7 @@ impl<'a> RequestDispatcher<'a> {
|
||||||
R::Params: DeserializeOwned + panic::UnwindSafe + Send + fmt::Debug,
|
R::Params: DeserializeOwned + panic::UnwindSafe + Send + fmt::Debug,
|
||||||
R::Result: Serialize,
|
R::Result: Serialize,
|
||||||
{
|
{
|
||||||
self.on_with_qos::<R>(QoSClass::Utility, f)
|
self.on_with_thread_intent::<R>(ThreadIntent::Worker, f)
|
||||||
}
|
}
|
||||||
|
|
||||||
/// Dispatches a latency-sensitive request onto the thread pool.
|
/// Dispatches a latency-sensitive request onto the thread pool.
|
||||||
|
@ -148,7 +148,7 @@ impl<'a> RequestDispatcher<'a> {
|
||||||
R::Params: DeserializeOwned + panic::UnwindSafe + Send + fmt::Debug,
|
R::Params: DeserializeOwned + panic::UnwindSafe + Send + fmt::Debug,
|
||||||
R::Result: Serialize,
|
R::Result: Serialize,
|
||||||
{
|
{
|
||||||
self.on_with_qos::<R>(QoSClass::UserInitiated, f)
|
self.on_with_thread_intent::<R>(ThreadIntent::LatencySensitive, f)
|
||||||
}
|
}
|
||||||
|
|
||||||
pub(crate) fn finish(&mut self) {
|
pub(crate) fn finish(&mut self) {
|
||||||
|
@ -163,9 +163,9 @@ impl<'a> RequestDispatcher<'a> {
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
|
|
||||||
fn on_with_qos<R>(
|
fn on_with_thread_intent<R>(
|
||||||
&mut self,
|
&mut self,
|
||||||
qos_class: QoSClass,
|
intent: ThreadIntent,
|
||||||
f: fn(GlobalStateSnapshot, R::Params) -> Result<R::Result>,
|
f: fn(GlobalStateSnapshot, R::Params) -> Result<R::Result>,
|
||||||
) -> &mut Self
|
) -> &mut Self
|
||||||
where
|
where
|
||||||
|
@ -178,7 +178,7 @@ impl<'a> RequestDispatcher<'a> {
|
||||||
None => return self,
|
None => return self,
|
||||||
};
|
};
|
||||||
|
|
||||||
self.global_state.task_pool.handle.spawn(qos_class, {
|
self.global_state.task_pool.handle.spawn(intent, {
|
||||||
let world = self.global_state.snapshot();
|
let world = self.global_state.snapshot();
|
||||||
move || {
|
move || {
|
||||||
let result = panic::catch_unwind(move || {
|
let result = panic::catch_unwind(move || {
|
||||||
|
|
|
@ -291,7 +291,7 @@ fn run_flycheck(state: &mut GlobalState, vfs_path: VfsPath) -> bool {
|
||||||
}
|
}
|
||||||
Ok(())
|
Ok(())
|
||||||
};
|
};
|
||||||
state.task_pool.handle.spawn_with_sender(stdx::thread::QoSClass::Utility, move |_| {
|
state.task_pool.handle.spawn_with_sender(stdx::thread::ThreadIntent::Worker, move |_| {
|
||||||
if let Err(e) = std::panic::catch_unwind(task) {
|
if let Err(e) = std::panic::catch_unwind(task) {
|
||||||
tracing::error!("flycheck task panicked: {e:?}")
|
tracing::error!("flycheck task panicked: {e:?}")
|
||||||
}
|
}
|
||||||
|
|
|
@ -397,7 +397,7 @@ impl GlobalState {
|
||||||
tracing::debug!(%cause, "will prime caches");
|
tracing::debug!(%cause, "will prime caches");
|
||||||
let num_worker_threads = self.config.prime_caches_num_threads();
|
let num_worker_threads = self.config.prime_caches_num_threads();
|
||||||
|
|
||||||
self.task_pool.handle.spawn_with_sender(stdx::thread::QoSClass::Utility, {
|
self.task_pool.handle.spawn_with_sender(stdx::thread::ThreadIntent::Worker, {
|
||||||
let analysis = self.snapshot().analysis;
|
let analysis = self.snapshot().analysis;
|
||||||
move |sender| {
|
move |sender| {
|
||||||
sender.send(Task::PrimeCaches(PrimeCachesProgress::Begin)).unwrap();
|
sender.send(Task::PrimeCaches(PrimeCachesProgress::Begin)).unwrap();
|
||||||
|
@ -680,7 +680,7 @@ impl GlobalState {
|
||||||
.on_sync::<lsp_ext::OnTypeFormatting>(handlers::handle_on_type_formatting)
|
.on_sync::<lsp_ext::OnTypeFormatting>(handlers::handle_on_type_formatting)
|
||||||
// We can’t run latency-sensitive request handlers which do semantic
|
// We can’t run latency-sensitive request handlers which do semantic
|
||||||
// analysis on the main thread because that would block other
|
// analysis on the main thread because that would block other
|
||||||
// requests. Instead, we run these request handlers on higher QoS
|
// requests. Instead, we run these request handlers on higher priority
|
||||||
// threads in the threadpool.
|
// threads in the threadpool.
|
||||||
.on_latency_sensitive::<lsp_types::request::Completion>(handlers::handle_completion)
|
.on_latency_sensitive::<lsp_types::request::Completion>(handlers::handle_completion)
|
||||||
.on_latency_sensitive::<lsp_types::request::ResolveCompletionItem>(
|
.on_latency_sensitive::<lsp_types::request::ResolveCompletionItem>(
|
||||||
|
@ -789,8 +789,8 @@ impl GlobalState {
|
||||||
let snapshot = self.snapshot();
|
let snapshot = self.snapshot();
|
||||||
|
|
||||||
// Diagnostics are triggered by the user typing
|
// Diagnostics are triggered by the user typing
|
||||||
// so we want computing them to run at the User Initiated QoS.
|
// so we run them on a latency sensitive thread.
|
||||||
self.task_pool.handle.spawn(stdx::thread::QoSClass::UserInitiated, move || {
|
self.task_pool.handle.spawn(stdx::thread::ThreadIntent::LatencySensitive, move || {
|
||||||
let _p = profile::span("publish_diagnostics");
|
let _p = profile::span("publish_diagnostics");
|
||||||
let diagnostics = subscriptions
|
let diagnostics = subscriptions
|
||||||
.into_iter()
|
.into_iter()
|
||||||
|
|
|
@ -27,7 +27,7 @@ use ide_db::{
|
||||||
use itertools::Itertools;
|
use itertools::Itertools;
|
||||||
use proc_macro_api::{MacroDylib, ProcMacroServer};
|
use proc_macro_api::{MacroDylib, ProcMacroServer};
|
||||||
use project_model::{PackageRoot, ProjectWorkspace, WorkspaceBuildScripts};
|
use project_model::{PackageRoot, ProjectWorkspace, WorkspaceBuildScripts};
|
||||||
use stdx::format_to;
|
use stdx::{format_to, thread::ThreadIntent};
|
||||||
use syntax::SmolStr;
|
use syntax::SmolStr;
|
||||||
use triomphe::Arc;
|
use triomphe::Arc;
|
||||||
use vfs::{file_set::FileSetConfig, AbsPath, AbsPathBuf, ChangeKind};
|
use vfs::{file_set::FileSetConfig, AbsPath, AbsPathBuf, ChangeKind};
|
||||||
|
@ -185,7 +185,7 @@ impl GlobalState {
|
||||||
pub(crate) fn fetch_workspaces(&mut self, cause: Cause) {
|
pub(crate) fn fetch_workspaces(&mut self, cause: Cause) {
|
||||||
tracing::info!(%cause, "will fetch workspaces");
|
tracing::info!(%cause, "will fetch workspaces");
|
||||||
|
|
||||||
self.task_pool.handle.spawn_with_sender(stdx::thread::QoSClass::Utility, {
|
self.task_pool.handle.spawn_with_sender(ThreadIntent::Worker, {
|
||||||
let linked_projects = self.config.linked_projects();
|
let linked_projects = self.config.linked_projects();
|
||||||
let detached_files = self.config.detached_files().to_vec();
|
let detached_files = self.config.detached_files().to_vec();
|
||||||
let cargo_config = self.config.cargo();
|
let cargo_config = self.config.cargo();
|
||||||
|
@ -260,7 +260,7 @@ impl GlobalState {
|
||||||
tracing::info!(%cause, "will fetch build data");
|
tracing::info!(%cause, "will fetch build data");
|
||||||
let workspaces = Arc::clone(&self.workspaces);
|
let workspaces = Arc::clone(&self.workspaces);
|
||||||
let config = self.config.cargo();
|
let config = self.config.cargo();
|
||||||
self.task_pool.handle.spawn_with_sender(stdx::thread::QoSClass::Utility, move |sender| {
|
self.task_pool.handle.spawn_with_sender(ThreadIntent::Worker, move |sender| {
|
||||||
sender.send(Task::FetchBuildData(BuildDataProgress::Begin)).unwrap();
|
sender.send(Task::FetchBuildData(BuildDataProgress::Begin)).unwrap();
|
||||||
|
|
||||||
let progress = {
|
let progress = {
|
||||||
|
@ -280,7 +280,7 @@ impl GlobalState {
|
||||||
let dummy_replacements = self.config.dummy_replacements().clone();
|
let dummy_replacements = self.config.dummy_replacements().clone();
|
||||||
let proc_macro_clients = self.proc_macro_clients.clone();
|
let proc_macro_clients = self.proc_macro_clients.clone();
|
||||||
|
|
||||||
self.task_pool.handle.spawn_with_sender(stdx::thread::QoSClass::Utility, move |sender| {
|
self.task_pool.handle.spawn_with_sender(ThreadIntent::Worker, move |sender| {
|
||||||
sender.send(Task::LoadProcMacros(ProcMacroProgress::Begin)).unwrap();
|
sender.send(Task::LoadProcMacros(ProcMacroProgress::Begin)).unwrap();
|
||||||
|
|
||||||
let dummy_replacements = &dummy_replacements;
|
let dummy_replacements = &dummy_replacements;
|
||||||
|
|
|
@ -2,7 +2,7 @@
|
||||||
//! It is used in [`crate::global_state::GlobalState`] throughout the main loop.
|
//! It is used in [`crate::global_state::GlobalState`] throughout the main loop.
|
||||||
|
|
||||||
use crossbeam_channel::Sender;
|
use crossbeam_channel::Sender;
|
||||||
use stdx::thread::{Pool, QoSClass};
|
use stdx::thread::{Pool, ThreadIntent};
|
||||||
|
|
||||||
pub(crate) struct TaskPool<T> {
|
pub(crate) struct TaskPool<T> {
|
||||||
sender: Sender<T>,
|
sender: Sender<T>,
|
||||||
|
@ -14,23 +14,23 @@ impl<T> TaskPool<T> {
|
||||||
TaskPool { sender, pool: Pool::new(threads) }
|
TaskPool { sender, pool: Pool::new(threads) }
|
||||||
}
|
}
|
||||||
|
|
||||||
pub(crate) fn spawn<F>(&mut self, qos_class: QoSClass, task: F)
|
pub(crate) fn spawn<F>(&mut self, intent: ThreadIntent, task: F)
|
||||||
where
|
where
|
||||||
F: FnOnce() -> T + Send + 'static,
|
F: FnOnce() -> T + Send + 'static,
|
||||||
T: Send + 'static,
|
T: Send + 'static,
|
||||||
{
|
{
|
||||||
self.pool.spawn(qos_class, {
|
self.pool.spawn(intent, {
|
||||||
let sender = self.sender.clone();
|
let sender = self.sender.clone();
|
||||||
move || sender.send(task()).unwrap()
|
move || sender.send(task()).unwrap()
|
||||||
})
|
})
|
||||||
}
|
}
|
||||||
|
|
||||||
pub(crate) fn spawn_with_sender<F>(&mut self, qos_class: QoSClass, task: F)
|
pub(crate) fn spawn_with_sender<F>(&mut self, intent: ThreadIntent, task: F)
|
||||||
where
|
where
|
||||||
F: FnOnce(Sender<T>) + Send + 'static,
|
F: FnOnce(Sender<T>) + Send + 'static,
|
||||||
T: Send + 'static,
|
T: Send + 'static,
|
||||||
{
|
{
|
||||||
self.pool.spawn(qos_class, {
|
self.pool.spawn(intent, {
|
||||||
let sender = self.sender.clone();
|
let sender = self.sender.clone();
|
||||||
move || task(sender)
|
move || task(sender)
|
||||||
})
|
})
|
||||||
|
|
|
@ -165,7 +165,7 @@ impl Server {
|
||||||
fn new(dir: TestDir, config: Config) -> Server {
|
fn new(dir: TestDir, config: Config) -> Server {
|
||||||
let (connection, client) = Connection::memory();
|
let (connection, client) = Connection::memory();
|
||||||
|
|
||||||
let _thread = stdx::thread::Builder::new(stdx::thread::QoSClass::Utility)
|
let _thread = stdx::thread::Builder::new(stdx::thread::ThreadIntent::Worker)
|
||||||
.name("test server".to_string())
|
.name("test server".to_string())
|
||||||
.spawn(move || main_loop(config, connection).unwrap())
|
.spawn(move || main_loop(config, connection).unwrap())
|
||||||
.expect("failed to spawn a thread");
|
.expect("failed to spawn a thread");
|
||||||
|
|
|
@ -1,39 +1,46 @@
|
||||||
//! A utility module for working with threads that automatically joins threads upon drop
|
//! A utility module for working with threads that automatically joins threads upon drop
|
||||||
//! and provides functionality for interfacing with operating system quality of service (QoS) APIs.
|
//! and abstracts over operating system quality of service (QoS) APIs
|
||||||
|
//! through the concept of a “thread intent”.
|
||||||
|
//!
|
||||||
|
//! The intent of a thread is frozen at thread creation time,
|
||||||
|
//! i.e. there is no API to change the intent of a thread once it has been spawned.
|
||||||
//!
|
//!
|
||||||
//! As a system, rust-analyzer should have the property that
|
//! As a system, rust-analyzer should have the property that
|
||||||
//! old manual scheduling APIs are replaced entirely by QoS.
|
//! old manual scheduling APIs are replaced entirely by QoS.
|
||||||
//! To maintain this invariant, we panic when it is clear that
|
//! To maintain this invariant, we panic when it is clear that
|
||||||
//! old scheduling APIs have been used.
|
//! old scheduling APIs have been used.
|
||||||
//!
|
//!
|
||||||
//! Moreover, we also want to ensure that every thread has a QoS set explicitly
|
//! Moreover, we also want to ensure that every thread has an intent set explicitly
|
||||||
//! to force a decision about its importance to the system.
|
//! to force a decision about its importance to the system.
|
||||||
//! Thus, [`QoSClass`] has no default value
|
//! Thus, [`ThreadIntent`] has no default value
|
||||||
//! and every entry point to creating a thread requires a [`QoSClass`] upfront.
|
//! and every entry point to creating a thread requires a [`ThreadIntent`] upfront.
|
||||||
|
|
||||||
use std::fmt;
|
use std::fmt;
|
||||||
|
|
||||||
|
mod intent;
|
||||||
mod pool;
|
mod pool;
|
||||||
|
|
||||||
|
pub use intent::ThreadIntent;
|
||||||
pub use pool::Pool;
|
pub use pool::Pool;
|
||||||
|
|
||||||
pub fn spawn<F, T>(qos_class: QoSClass, f: F) -> JoinHandle<T>
|
pub fn spawn<F, T>(intent: ThreadIntent, f: F) -> JoinHandle<T>
|
||||||
where
|
where
|
||||||
F: FnOnce() -> T,
|
F: FnOnce() -> T,
|
||||||
F: Send + 'static,
|
F: Send + 'static,
|
||||||
T: Send + 'static,
|
T: Send + 'static,
|
||||||
{
|
{
|
||||||
Builder::new(qos_class).spawn(f).expect("failed to spawn thread")
|
Builder::new(intent).spawn(f).expect("failed to spawn thread")
|
||||||
}
|
}
|
||||||
|
|
||||||
pub struct Builder {
|
pub struct Builder {
|
||||||
qos_class: QoSClass,
|
intent: ThreadIntent,
|
||||||
inner: jod_thread::Builder,
|
inner: jod_thread::Builder,
|
||||||
allow_leak: bool,
|
allow_leak: bool,
|
||||||
}
|
}
|
||||||
|
|
||||||
impl Builder {
|
impl Builder {
|
||||||
pub fn new(qos_class: QoSClass) -> Builder {
|
pub fn new(intent: ThreadIntent) -> Builder {
|
||||||
Builder { qos_class, inner: jod_thread::Builder::new(), allow_leak: false }
|
Builder { intent, inner: jod_thread::Builder::new(), allow_leak: false }
|
||||||
}
|
}
|
||||||
|
|
||||||
pub fn name(self, name: String) -> Builder {
|
pub fn name(self, name: String) -> Builder {
|
||||||
|
@ -55,7 +62,7 @@ impl Builder {
|
||||||
T: Send + 'static,
|
T: Send + 'static,
|
||||||
{
|
{
|
||||||
let inner_handle = self.inner.spawn(move || {
|
let inner_handle = self.inner.spawn(move || {
|
||||||
set_current_thread_qos_class(self.qos_class);
|
self.intent.apply_to_current_thread();
|
||||||
f()
|
f()
|
||||||
})?;
|
})?;
|
||||||
|
|
||||||
|
@ -93,237 +100,3 @@ impl<T> fmt::Debug for JoinHandle<T> {
|
||||||
f.pad("JoinHandle { .. }")
|
f.pad("JoinHandle { .. }")
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
|
|
||||||
#[derive(Debug, Clone, Copy, PartialEq, Eq, PartialOrd, Ord)]
|
|
||||||
// Please maintain order from least to most priority for the derived `Ord` impl.
|
|
||||||
pub enum QoSClass {
|
|
||||||
// Documentation adapted from https://github.com/apple-oss-distributions/libpthread/blob/67e155c94093be9a204b69637d198eceff2c7c46/include/sys/qos.h#L55
|
|
||||||
//
|
|
||||||
/// TLDR: invisible maintenance tasks
|
|
||||||
///
|
|
||||||
/// Contract:
|
|
||||||
///
|
|
||||||
/// * **You do not care about how long it takes for work to finish.**
|
|
||||||
/// * **You do not care about work being deferred temporarily.**
|
|
||||||
/// (e.g. if the device’s battery is in a critical state)
|
|
||||||
///
|
|
||||||
/// Examples:
|
|
||||||
///
|
|
||||||
/// * in a video editor:
|
|
||||||
/// creating periodic backups of project files
|
|
||||||
/// * in a browser:
|
|
||||||
/// cleaning up cached sites which have not been accessed in a long time
|
|
||||||
/// * in a collaborative word processor:
|
|
||||||
/// creating a searchable index of all documents
|
|
||||||
///
|
|
||||||
/// Use this QoS class for background tasks
|
|
||||||
/// which the user did not initiate themselves
|
|
||||||
/// and which are invisible to the user.
|
|
||||||
/// It is expected that this work will take significant time to complete:
|
|
||||||
/// minutes or even hours.
|
|
||||||
///
|
|
||||||
/// This QoS class provides the most energy and thermally-efficient execution possible.
|
|
||||||
/// All other work is prioritized over background tasks.
|
|
||||||
Background,
|
|
||||||
|
|
||||||
/// TLDR: tasks that don’t block using your app
|
|
||||||
///
|
|
||||||
/// Contract:
|
|
||||||
///
|
|
||||||
/// * **Your app remains useful even as the task is executing.**
|
|
||||||
///
|
|
||||||
/// Examples:
|
|
||||||
///
|
|
||||||
/// * in a video editor:
|
|
||||||
/// exporting a video to disk –
|
|
||||||
/// the user can still work on the timeline
|
|
||||||
/// * in a browser:
|
|
||||||
/// automatically extracting a downloaded zip file –
|
|
||||||
/// the user can still switch tabs
|
|
||||||
/// * in a collaborative word processor:
|
|
||||||
/// downloading images embedded in a document –
|
|
||||||
/// the user can still make edits
|
|
||||||
///
|
|
||||||
/// Use this QoS class for tasks which
|
|
||||||
/// may or may not be initiated by the user,
|
|
||||||
/// but whose result is visible.
|
|
||||||
/// It is expected that this work will take a few seconds to a few minutes.
|
|
||||||
/// Typically your app will include a progress bar
|
|
||||||
/// for tasks using this class.
|
|
||||||
///
|
|
||||||
/// This QoS class provides a balance between
|
|
||||||
/// performance, responsiveness and efficiency.
|
|
||||||
Utility,
|
|
||||||
|
|
||||||
/// TLDR: tasks that block using your app
|
|
||||||
///
|
|
||||||
/// Contract:
|
|
||||||
///
|
|
||||||
/// * **You need this work to complete
|
|
||||||
/// before the user can keep interacting with your app.**
|
|
||||||
/// * **Your work will not take more than a few seconds to complete.**
|
|
||||||
///
|
|
||||||
/// Examples:
|
|
||||||
///
|
|
||||||
/// * in a video editor:
|
|
||||||
/// opening a saved project
|
|
||||||
/// * in a browser:
|
|
||||||
/// loading a list of the user’s bookmarks and top sites
|
|
||||||
/// when a new tab is created
|
|
||||||
/// * in a collaborative word processor:
|
|
||||||
/// running a search on the document’s content
|
|
||||||
///
|
|
||||||
/// Use this QoS class for tasks which were initiated by the user
|
|
||||||
/// and block the usage of your app while they are in progress.
|
|
||||||
/// It is expected that this work will take a few seconds or less to complete;
|
|
||||||
/// not long enough to cause the user to switch to something else.
|
|
||||||
/// Your app will likely indicate progress on these tasks
|
|
||||||
/// through the display of placeholder content or modals.
|
|
||||||
///
|
|
||||||
/// This QoS class is not energy-efficient.
|
|
||||||
/// Rather, it provides responsiveness
|
|
||||||
/// by prioritizing work above other tasks on the system
|
|
||||||
/// except for critical user-interactive work.
|
|
||||||
UserInitiated,
|
|
||||||
|
|
||||||
/// TLDR: render loops and nothing else
|
|
||||||
///
|
|
||||||
/// Contract:
|
|
||||||
///
|
|
||||||
/// * **You absolutely need this work to complete immediately
|
|
||||||
/// or your app will appear to freeze.**
|
|
||||||
/// * **Your work will always complete virtually instantaneously.**
|
|
||||||
///
|
|
||||||
/// Examples:
|
|
||||||
///
|
|
||||||
/// * the main thread in a GUI application
|
|
||||||
/// * the update & render loop in a game
|
|
||||||
/// * a secondary thread which progresses an animation
|
|
||||||
///
|
|
||||||
/// Use this QoS class for any work which, if delayed,
|
|
||||||
/// will make your user interface unresponsive.
|
|
||||||
/// It is expected that this work will be virtually instantaneous.
|
|
||||||
///
|
|
||||||
/// This QoS class is not energy-efficient.
|
|
||||||
/// Specifying this class is a request to run with
|
|
||||||
/// nearly all available system CPU and I/O bandwidth even under contention.
|
|
||||||
UserInteractive,
|
|
||||||
}
|
|
||||||
|
|
||||||
pub const IS_QOS_AVAILABLE: bool = imp::IS_QOS_AVAILABLE;
|
|
||||||
|
|
||||||
pub fn set_current_thread_qos_class(class: QoSClass) {
|
|
||||||
imp::set_current_thread_qos_class(class)
|
|
||||||
}
|
|
||||||
|
|
||||||
pub fn get_current_thread_qos_class() -> Option<QoSClass> {
|
|
||||||
imp::get_current_thread_qos_class()
|
|
||||||
}
|
|
||||||
|
|
||||||
// All Apple platforms use XNU as their kernel
|
|
||||||
// and thus have the concept of QoS.
|
|
||||||
#[cfg(target_vendor = "apple")]
|
|
||||||
mod imp {
|
|
||||||
use super::QoSClass;
|
|
||||||
|
|
||||||
pub(super) const IS_QOS_AVAILABLE: bool = true;
|
|
||||||
|
|
||||||
pub(super) fn set_current_thread_qos_class(class: QoSClass) {
|
|
||||||
let c = match class {
|
|
||||||
QoSClass::UserInteractive => libc::qos_class_t::QOS_CLASS_USER_INTERACTIVE,
|
|
||||||
QoSClass::UserInitiated => libc::qos_class_t::QOS_CLASS_USER_INITIATED,
|
|
||||||
QoSClass::Utility => libc::qos_class_t::QOS_CLASS_UTILITY,
|
|
||||||
QoSClass::Background => libc::qos_class_t::QOS_CLASS_BACKGROUND,
|
|
||||||
};
|
|
||||||
|
|
||||||
let code = unsafe { libc::pthread_set_qos_class_self_np(c, 0) };
|
|
||||||
|
|
||||||
if code == 0 {
|
|
||||||
return;
|
|
||||||
}
|
|
||||||
|
|
||||||
let errno = unsafe { *libc::__error() };
|
|
||||||
|
|
||||||
match errno {
|
|
||||||
libc::EPERM => {
|
|
||||||
// This thread has been excluded from the QoS system
|
|
||||||
// due to a previous call to a function such as `pthread_setschedparam`
|
|
||||||
// which is incompatible with QoS.
|
|
||||||
//
|
|
||||||
// Panic instead of returning an error
|
|
||||||
// to maintain the invariant that we only use QoS APIs.
|
|
||||||
panic!("tried to set QoS of thread which has opted out of QoS (os error {errno})")
|
|
||||||
}
|
|
||||||
|
|
||||||
libc::EINVAL => {
|
|
||||||
// This is returned if we pass something other than a qos_class_t
|
|
||||||
// to `pthread_set_qos_class_self_np`.
|
|
||||||
//
|
|
||||||
// This is impossible, so again panic.
|
|
||||||
unreachable!(
|
|
||||||
"invalid qos_class_t value was passed to pthread_set_qos_class_self_np"
|
|
||||||
)
|
|
||||||
}
|
|
||||||
|
|
||||||
_ => {
|
|
||||||
// `pthread_set_qos_class_self_np`’s documentation
|
|
||||||
// does not mention any other errors.
|
|
||||||
unreachable!("`pthread_set_qos_class_self_np` returned unexpected error {errno}")
|
|
||||||
}
|
|
||||||
}
|
|
||||||
}
|
|
||||||
|
|
||||||
pub(super) fn get_current_thread_qos_class() -> Option<QoSClass> {
|
|
||||||
let current_thread = unsafe { libc::pthread_self() };
|
|
||||||
let mut qos_class_raw = libc::qos_class_t::QOS_CLASS_UNSPECIFIED;
|
|
||||||
let code = unsafe {
|
|
||||||
libc::pthread_get_qos_class_np(current_thread, &mut qos_class_raw, std::ptr::null_mut())
|
|
||||||
};
|
|
||||||
|
|
||||||
if code != 0 {
|
|
||||||
// `pthread_get_qos_class_np`’s documentation states that
|
|
||||||
// an error value is placed into errno if the return code is not zero.
|
|
||||||
// However, it never states what errors are possible.
|
|
||||||
// Inspecting the source[0] shows that, as of this writing, it always returns zero.
|
|
||||||
//
|
|
||||||
// Whatever errors the function could report in future are likely to be
|
|
||||||
// ones which we cannot handle anyway
|
|
||||||
//
|
|
||||||
// 0: https://github.com/apple-oss-distributions/libpthread/blob/67e155c94093be9a204b69637d198eceff2c7c46/src/qos.c#L171-L177
|
|
||||||
let errno = unsafe { *libc::__error() };
|
|
||||||
unreachable!("`pthread_get_qos_class_np` failed unexpectedly (os error {errno})");
|
|
||||||
}
|
|
||||||
|
|
||||||
match qos_class_raw {
|
|
||||||
libc::qos_class_t::QOS_CLASS_USER_INTERACTIVE => Some(QoSClass::UserInteractive),
|
|
||||||
libc::qos_class_t::QOS_CLASS_USER_INITIATED => Some(QoSClass::UserInitiated),
|
|
||||||
libc::qos_class_t::QOS_CLASS_DEFAULT => None, // QoS has never been set
|
|
||||||
libc::qos_class_t::QOS_CLASS_UTILITY => Some(QoSClass::Utility),
|
|
||||||
libc::qos_class_t::QOS_CLASS_BACKGROUND => Some(QoSClass::Background),
|
|
||||||
|
|
||||||
libc::qos_class_t::QOS_CLASS_UNSPECIFIED => {
|
|
||||||
// Using manual scheduling APIs causes threads to “opt out” of QoS.
|
|
||||||
// At this point they become incompatible with QoS,
|
|
||||||
// and as such have the “unspecified” QoS class.
|
|
||||||
//
|
|
||||||
// Panic instead of returning an error
|
|
||||||
// to maintain the invariant that we only use QoS APIs.
|
|
||||||
panic!("tried to get QoS of thread which has opted out of QoS")
|
|
||||||
}
|
|
||||||
}
|
|
||||||
}
|
|
||||||
}
|
|
||||||
|
|
||||||
// FIXME: Windows has QoS APIs, we should use them!
|
|
||||||
#[cfg(not(target_vendor = "apple"))]
|
|
||||||
mod imp {
|
|
||||||
use super::QoSClass;
|
|
||||||
|
|
||||||
pub(super) const IS_QOS_AVAILABLE: bool = false;
|
|
||||||
|
|
||||||
pub(super) fn set_current_thread_qos_class(_: QoSClass) {}
|
|
||||||
|
|
||||||
pub(super) fn get_current_thread_qos_class() -> Option<QoSClass> {
|
|
||||||
None
|
|
||||||
}
|
|
||||||
}
|
|
||||||
|
|
287
crates/stdx/src/thread/intent.rs
Normal file
287
crates/stdx/src/thread/intent.rs
Normal file
|
@ -0,0 +1,287 @@
|
||||||
|
//! An opaque façade around platform-specific QoS APIs.
|
||||||
|
|
||||||
|
#[derive(Debug, Clone, Copy, PartialEq, Eq, PartialOrd, Ord)]
|
||||||
|
// Please maintain order from least to most priority for the derived `Ord` impl.
|
||||||
|
pub enum ThreadIntent {
|
||||||
|
/// Any thread which does work that isn’t in the critical path of the user typing
|
||||||
|
/// (e.g. processing Go To Definition).
|
||||||
|
Worker,
|
||||||
|
|
||||||
|
/// Any thread which does work caused by the user typing
|
||||||
|
/// (e.g. processing syntax highlighting).
|
||||||
|
LatencySensitive,
|
||||||
|
}
|
||||||
|
|
||||||
|
impl ThreadIntent {
|
||||||
|
// These APIs must remain private;
|
||||||
|
// we only want consumers to set thread intent
|
||||||
|
// either during thread creation or using our pool impl.
|
||||||
|
|
||||||
|
pub(super) fn apply_to_current_thread(self) {
|
||||||
|
let class = thread_intent_to_qos_class(self);
|
||||||
|
set_current_thread_qos_class(class);
|
||||||
|
}
|
||||||
|
|
||||||
|
pub(super) fn assert_is_used_on_current_thread(self) {
|
||||||
|
if IS_QOS_AVAILABLE {
|
||||||
|
let class = thread_intent_to_qos_class(self);
|
||||||
|
assert_eq!(get_current_thread_qos_class(), Some(class));
|
||||||
|
}
|
||||||
|
}
|
||||||
|
}
|
||||||
|
|
||||||
|
use imp::QoSClass;
|
||||||
|
|
||||||
|
const IS_QOS_AVAILABLE: bool = imp::IS_QOS_AVAILABLE;
|
||||||
|
|
||||||
|
fn set_current_thread_qos_class(class: QoSClass) {
|
||||||
|
imp::set_current_thread_qos_class(class)
|
||||||
|
}
|
||||||
|
|
||||||
|
fn get_current_thread_qos_class() -> Option<QoSClass> {
|
||||||
|
imp::get_current_thread_qos_class()
|
||||||
|
}
|
||||||
|
|
||||||
|
fn thread_intent_to_qos_class(intent: ThreadIntent) -> QoSClass {
|
||||||
|
imp::thread_intent_to_qos_class(intent)
|
||||||
|
}
|
||||||
|
|
||||||
|
// All Apple platforms use XNU as their kernel
|
||||||
|
// and thus have the concept of QoS.
|
||||||
|
#[cfg(target_vendor = "apple")]
|
||||||
|
mod imp {
|
||||||
|
use super::ThreadIntent;
|
||||||
|
|
||||||
|
#[derive(Debug, Clone, Copy, PartialEq, Eq, PartialOrd, Ord)]
|
||||||
|
// Please maintain order from least to most priority for the derived `Ord` impl.
|
||||||
|
pub(super) enum QoSClass {
|
||||||
|
// Documentation adapted from https://github.com/apple-oss-distributions/libpthread/blob/67e155c94093be9a204b69637d198eceff2c7c46/include/sys/qos.h#L55
|
||||||
|
//
|
||||||
|
/// TLDR: invisible maintenance tasks
|
||||||
|
///
|
||||||
|
/// Contract:
|
||||||
|
///
|
||||||
|
/// * **You do not care about how long it takes for work to finish.**
|
||||||
|
/// * **You do not care about work being deferred temporarily.**
|
||||||
|
/// (e.g. if the device’s battery is in a critical state)
|
||||||
|
///
|
||||||
|
/// Examples:
|
||||||
|
///
|
||||||
|
/// * in a video editor:
|
||||||
|
/// creating periodic backups of project files
|
||||||
|
/// * in a browser:
|
||||||
|
/// cleaning up cached sites which have not been accessed in a long time
|
||||||
|
/// * in a collaborative word processor:
|
||||||
|
/// creating a searchable index of all documents
|
||||||
|
///
|
||||||
|
/// Use this QoS class for background tasks
|
||||||
|
/// which the user did not initiate themselves
|
||||||
|
/// and which are invisible to the user.
|
||||||
|
/// It is expected that this work will take significant time to complete:
|
||||||
|
/// minutes or even hours.
|
||||||
|
///
|
||||||
|
/// This QoS class provides the most energy and thermally-efficient execution possible.
|
||||||
|
/// All other work is prioritized over background tasks.
|
||||||
|
Background,
|
||||||
|
|
||||||
|
/// TLDR: tasks that don’t block using your app
|
||||||
|
///
|
||||||
|
/// Contract:
|
||||||
|
///
|
||||||
|
/// * **Your app remains useful even as the task is executing.**
|
||||||
|
///
|
||||||
|
/// Examples:
|
||||||
|
///
|
||||||
|
/// * in a video editor:
|
||||||
|
/// exporting a video to disk –
|
||||||
|
/// the user can still work on the timeline
|
||||||
|
/// * in a browser:
|
||||||
|
/// automatically extracting a downloaded zip file –
|
||||||
|
/// the user can still switch tabs
|
||||||
|
/// * in a collaborative word processor:
|
||||||
|
/// downloading images embedded in a document –
|
||||||
|
/// the user can still make edits
|
||||||
|
///
|
||||||
|
/// Use this QoS class for tasks which
|
||||||
|
/// may or may not be initiated by the user,
|
||||||
|
/// but whose result is visible.
|
||||||
|
/// It is expected that this work will take a few seconds to a few minutes.
|
||||||
|
/// Typically your app will include a progress bar
|
||||||
|
/// for tasks using this class.
|
||||||
|
///
|
||||||
|
/// This QoS class provides a balance between
|
||||||
|
/// performance, responsiveness and efficiency.
|
||||||
|
Utility,
|
||||||
|
|
||||||
|
/// TLDR: tasks that block using your app
|
||||||
|
///
|
||||||
|
/// Contract:
|
||||||
|
///
|
||||||
|
/// * **You need this work to complete
|
||||||
|
/// before the user can keep interacting with your app.**
|
||||||
|
/// * **Your work will not take more than a few seconds to complete.**
|
||||||
|
///
|
||||||
|
/// Examples:
|
||||||
|
///
|
||||||
|
/// * in a video editor:
|
||||||
|
/// opening a saved project
|
||||||
|
/// * in a browser:
|
||||||
|
/// loading a list of the user’s bookmarks and top sites
|
||||||
|
/// when a new tab is created
|
||||||
|
/// * in a collaborative word processor:
|
||||||
|
/// running a search on the document’s content
|
||||||
|
///
|
||||||
|
/// Use this QoS class for tasks which were initiated by the user
|
||||||
|
/// and block the usage of your app while they are in progress.
|
||||||
|
/// It is expected that this work will take a few seconds or less to complete;
|
||||||
|
/// not long enough to cause the user to switch to something else.
|
||||||
|
/// Your app will likely indicate progress on these tasks
|
||||||
|
/// through the display of placeholder content or modals.
|
||||||
|
///
|
||||||
|
/// This QoS class is not energy-efficient.
|
||||||
|
/// Rather, it provides responsiveness
|
||||||
|
/// by prioritizing work above other tasks on the system
|
||||||
|
/// except for critical user-interactive work.
|
||||||
|
UserInitiated,
|
||||||
|
|
||||||
|
/// TLDR: render loops and nothing else
|
||||||
|
///
|
||||||
|
/// Contract:
|
||||||
|
///
|
||||||
|
/// * **You absolutely need this work to complete immediately
|
||||||
|
/// or your app will appear to freeze.**
|
||||||
|
/// * **Your work will always complete virtually instantaneously.**
|
||||||
|
///
|
||||||
|
/// Examples:
|
||||||
|
///
|
||||||
|
/// * the main thread in a GUI application
|
||||||
|
/// * the update & render loop in a game
|
||||||
|
/// * a secondary thread which progresses an animation
|
||||||
|
///
|
||||||
|
/// Use this QoS class for any work which, if delayed,
|
||||||
|
/// will make your user interface unresponsive.
|
||||||
|
/// It is expected that this work will be virtually instantaneous.
|
||||||
|
///
|
||||||
|
/// This QoS class is not energy-efficient.
|
||||||
|
/// Specifying this class is a request to run with
|
||||||
|
/// nearly all available system CPU and I/O bandwidth even under contention.
|
||||||
|
UserInteractive,
|
||||||
|
}
|
||||||
|
|
||||||
|
pub(super) const IS_QOS_AVAILABLE: bool = true;
|
||||||
|
|
||||||
|
pub(super) fn set_current_thread_qos_class(class: QoSClass) {
|
||||||
|
let c = match class {
|
||||||
|
QoSClass::UserInteractive => libc::qos_class_t::QOS_CLASS_USER_INTERACTIVE,
|
||||||
|
QoSClass::UserInitiated => libc::qos_class_t::QOS_CLASS_USER_INITIATED,
|
||||||
|
QoSClass::Utility => libc::qos_class_t::QOS_CLASS_UTILITY,
|
||||||
|
QoSClass::Background => libc::qos_class_t::QOS_CLASS_BACKGROUND,
|
||||||
|
};
|
||||||
|
|
||||||
|
let code = unsafe { libc::pthread_set_qos_class_self_np(c, 0) };
|
||||||
|
|
||||||
|
if code == 0 {
|
||||||
|
return;
|
||||||
|
}
|
||||||
|
|
||||||
|
let errno = unsafe { *libc::__error() };
|
||||||
|
|
||||||
|
match errno {
|
||||||
|
libc::EPERM => {
|
||||||
|
// This thread has been excluded from the QoS system
|
||||||
|
// due to a previous call to a function such as `pthread_setschedparam`
|
||||||
|
// which is incompatible with QoS.
|
||||||
|
//
|
||||||
|
// Panic instead of returning an error
|
||||||
|
// to maintain the invariant that we only use QoS APIs.
|
||||||
|
panic!("tried to set QoS of thread which has opted out of QoS (os error {errno})")
|
||||||
|
}
|
||||||
|
|
||||||
|
libc::EINVAL => {
|
||||||
|
// This is returned if we pass something other than a qos_class_t
|
||||||
|
// to `pthread_set_qos_class_self_np`.
|
||||||
|
//
|
||||||
|
// This is impossible, so again panic.
|
||||||
|
unreachable!(
|
||||||
|
"invalid qos_class_t value was passed to pthread_set_qos_class_self_np"
|
||||||
|
)
|
||||||
|
}
|
||||||
|
|
||||||
|
_ => {
|
||||||
|
// `pthread_set_qos_class_self_np`’s documentation
|
||||||
|
// does not mention any other errors.
|
||||||
|
unreachable!("`pthread_set_qos_class_self_np` returned unexpected error {errno}")
|
||||||
|
}
|
||||||
|
}
|
||||||
|
}
|
||||||
|
|
||||||
|
pub(super) fn get_current_thread_qos_class() -> Option<QoSClass> {
|
||||||
|
let current_thread = unsafe { libc::pthread_self() };
|
||||||
|
let mut qos_class_raw = libc::qos_class_t::QOS_CLASS_UNSPECIFIED;
|
||||||
|
let code = unsafe {
|
||||||
|
libc::pthread_get_qos_class_np(current_thread, &mut qos_class_raw, std::ptr::null_mut())
|
||||||
|
};
|
||||||
|
|
||||||
|
if code != 0 {
|
||||||
|
// `pthread_get_qos_class_np`’s documentation states that
|
||||||
|
// an error value is placed into errno if the return code is not zero.
|
||||||
|
// However, it never states what errors are possible.
|
||||||
|
// Inspecting the source[0] shows that, as of this writing, it always returns zero.
|
||||||
|
//
|
||||||
|
// Whatever errors the function could report in future are likely to be
|
||||||
|
// ones which we cannot handle anyway
|
||||||
|
//
|
||||||
|
// 0: https://github.com/apple-oss-distributions/libpthread/blob/67e155c94093be9a204b69637d198eceff2c7c46/src/qos.c#L171-L177
|
||||||
|
let errno = unsafe { *libc::__error() };
|
||||||
|
unreachable!("`pthread_get_qos_class_np` failed unexpectedly (os error {errno})");
|
||||||
|
}
|
||||||
|
|
||||||
|
match qos_class_raw {
|
||||||
|
libc::qos_class_t::QOS_CLASS_USER_INTERACTIVE => Some(QoSClass::UserInteractive),
|
||||||
|
libc::qos_class_t::QOS_CLASS_USER_INITIATED => Some(QoSClass::UserInitiated),
|
||||||
|
libc::qos_class_t::QOS_CLASS_DEFAULT => None, // QoS has never been set
|
||||||
|
libc::qos_class_t::QOS_CLASS_UTILITY => Some(QoSClass::Utility),
|
||||||
|
libc::qos_class_t::QOS_CLASS_BACKGROUND => Some(QoSClass::Background),
|
||||||
|
|
||||||
|
libc::qos_class_t::QOS_CLASS_UNSPECIFIED => {
|
||||||
|
// Using manual scheduling APIs causes threads to “opt out” of QoS.
|
||||||
|
// At this point they become incompatible with QoS,
|
||||||
|
// and as such have the “unspecified” QoS class.
|
||||||
|
//
|
||||||
|
// Panic instead of returning an error
|
||||||
|
// to maintain the invariant that we only use QoS APIs.
|
||||||
|
panic!("tried to get QoS of thread which has opted out of QoS")
|
||||||
|
}
|
||||||
|
}
|
||||||
|
}
|
||||||
|
|
||||||
|
pub(super) fn thread_intent_to_qos_class(intent: ThreadIntent) -> QoSClass {
|
||||||
|
match intent {
|
||||||
|
ThreadIntent::Worker => QoSClass::Utility,
|
||||||
|
ThreadIntent::LatencySensitive => QoSClass::UserInitiated,
|
||||||
|
}
|
||||||
|
}
|
||||||
|
}
|
||||||
|
|
||||||
|
// FIXME: Windows has QoS APIs, we should use them!
|
||||||
|
#[cfg(not(target_vendor = "apple"))]
|
||||||
|
mod imp {
|
||||||
|
use super::ThreadIntent;
|
||||||
|
|
||||||
|
#[derive(Debug, Clone, Copy, PartialEq, Eq, PartialOrd, Ord)]
|
||||||
|
pub(super) enum QoSClass {
|
||||||
|
Default,
|
||||||
|
}
|
||||||
|
|
||||||
|
pub(super) const IS_QOS_AVAILABLE: bool = false;
|
||||||
|
|
||||||
|
pub(super) fn set_current_thread_qos_class(_: QoSClass) {}
|
||||||
|
|
||||||
|
pub(super) fn get_current_thread_qos_class() -> Option<QoSClass> {
|
||||||
|
None
|
||||||
|
}
|
||||||
|
|
||||||
|
pub(super) fn thread_intent_to_qos_class(_: ThreadIntent) -> QoSClass {
|
||||||
|
QoSClass::Default
|
||||||
|
}
|
||||||
|
}
|
|
@ -1,6 +1,7 @@
|
||||||
//! [`Pool`] implements a basic custom thread pool
|
//! [`Pool`] implements a basic custom thread pool
|
||||||
//! inspired by the [`threadpool` crate](http://docs.rs/threadpool).
|
//! inspired by the [`threadpool` crate](http://docs.rs/threadpool).
|
||||||
//! It allows the spawning of tasks under different QoS classes.
|
//! When you spawn a task you specify a thread intent
|
||||||
|
//! so the pool can schedule it to run on a thread with that intent.
|
||||||
//! rust-analyzer uses this to prioritize work based on latency requirements.
|
//! rust-analyzer uses this to prioritize work based on latency requirements.
|
||||||
//!
|
//!
|
||||||
//! The thread pool is implemented entirely using
|
//! The thread pool is implemented entirely using
|
||||||
|
@ -13,10 +14,7 @@ use std::sync::{
|
||||||
|
|
||||||
use crossbeam_channel::{Receiver, Sender};
|
use crossbeam_channel::{Receiver, Sender};
|
||||||
|
|
||||||
use super::{
|
use super::{Builder, JoinHandle, ThreadIntent};
|
||||||
get_current_thread_qos_class, set_current_thread_qos_class, Builder, JoinHandle, QoSClass,
|
|
||||||
IS_QOS_AVAILABLE,
|
|
||||||
};
|
|
||||||
|
|
||||||
pub struct Pool {
|
pub struct Pool {
|
||||||
// `_handles` is never read: the field is present
|
// `_handles` is never read: the field is present
|
||||||
|
@ -32,32 +30,32 @@ pub struct Pool {
|
||||||
}
|
}
|
||||||
|
|
||||||
struct Job {
|
struct Job {
|
||||||
requested_qos_class: QoSClass,
|
requested_intent: ThreadIntent,
|
||||||
f: Box<dyn FnOnce() + Send + 'static>,
|
f: Box<dyn FnOnce() + Send + 'static>,
|
||||||
}
|
}
|
||||||
|
|
||||||
impl Pool {
|
impl Pool {
|
||||||
pub fn new(threads: usize) -> Pool {
|
pub fn new(threads: usize) -> Pool {
|
||||||
const STACK_SIZE: usize = 8 * 1024 * 1024;
|
const STACK_SIZE: usize = 8 * 1024 * 1024;
|
||||||
const INITIAL_QOS_CLASS: QoSClass = QoSClass::Utility;
|
const INITIAL_INTENT: ThreadIntent = ThreadIntent::Worker;
|
||||||
|
|
||||||
let (job_sender, job_receiver) = crossbeam_channel::unbounded();
|
let (job_sender, job_receiver) = crossbeam_channel::unbounded();
|
||||||
let extant_tasks = Arc::new(AtomicUsize::new(0));
|
let extant_tasks = Arc::new(AtomicUsize::new(0));
|
||||||
|
|
||||||
let mut handles = Vec::with_capacity(threads);
|
let mut handles = Vec::with_capacity(threads);
|
||||||
for _ in 0..threads {
|
for _ in 0..threads {
|
||||||
let handle = Builder::new(INITIAL_QOS_CLASS)
|
let handle = Builder::new(INITIAL_INTENT)
|
||||||
.stack_size(STACK_SIZE)
|
.stack_size(STACK_SIZE)
|
||||||
.name("Worker".into())
|
.name("Worker".into())
|
||||||
.spawn({
|
.spawn({
|
||||||
let extant_tasks = Arc::clone(&extant_tasks);
|
let extant_tasks = Arc::clone(&extant_tasks);
|
||||||
let job_receiver: Receiver<Job> = job_receiver.clone();
|
let job_receiver: Receiver<Job> = job_receiver.clone();
|
||||||
move || {
|
move || {
|
||||||
let mut current_qos_class = INITIAL_QOS_CLASS;
|
let mut current_intent = INITIAL_INTENT;
|
||||||
for job in job_receiver {
|
for job in job_receiver {
|
||||||
if job.requested_qos_class != current_qos_class {
|
if job.requested_intent != current_intent {
|
||||||
set_current_thread_qos_class(job.requested_qos_class);
|
job.requested_intent.apply_to_current_thread();
|
||||||
current_qos_class = job.requested_qos_class;
|
current_intent = job.requested_intent;
|
||||||
}
|
}
|
||||||
extant_tasks.fetch_add(1, Ordering::SeqCst);
|
extant_tasks.fetch_add(1, Ordering::SeqCst);
|
||||||
(job.f)();
|
(job.f)();
|
||||||
|
@ -73,19 +71,18 @@ impl Pool {
|
||||||
Pool { _handles: handles, extant_tasks, job_sender }
|
Pool { _handles: handles, extant_tasks, job_sender }
|
||||||
}
|
}
|
||||||
|
|
||||||
pub fn spawn<F>(&self, qos_class: QoSClass, f: F)
|
pub fn spawn<F>(&self, intent: ThreadIntent, f: F)
|
||||||
where
|
where
|
||||||
F: FnOnce() + Send + 'static,
|
F: FnOnce() + Send + 'static,
|
||||||
{
|
{
|
||||||
let f = Box::new(move || {
|
let f = Box::new(move || {
|
||||||
if IS_QOS_AVAILABLE {
|
if cfg!(debug_assertions) {
|
||||||
debug_assert_eq!(get_current_thread_qos_class(), Some(qos_class));
|
intent.assert_is_used_on_current_thread();
|
||||||
}
|
}
|
||||||
|
|
||||||
f()
|
f()
|
||||||
});
|
});
|
||||||
|
|
||||||
let job = Job { requested_qos_class: qos_class, f };
|
let job = Job { requested_intent: intent, f };
|
||||||
self.job_sender.send(job).unwrap();
|
self.job_sender.send(job).unwrap();
|
||||||
}
|
}
|
||||||
|
|
||||||
|
|
|
@ -34,7 +34,7 @@ impl loader::Handle for NotifyHandle {
|
||||||
fn spawn(sender: loader::Sender) -> NotifyHandle {
|
fn spawn(sender: loader::Sender) -> NotifyHandle {
|
||||||
let actor = NotifyActor::new(sender);
|
let actor = NotifyActor::new(sender);
|
||||||
let (sender, receiver) = unbounded::<Message>();
|
let (sender, receiver) = unbounded::<Message>();
|
||||||
let thread = stdx::thread::Builder::new(stdx::thread::QoSClass::Utility)
|
let thread = stdx::thread::Builder::new(stdx::thread::ThreadIntent::Worker)
|
||||||
.name("VfsLoader".to_owned())
|
.name("VfsLoader".to_owned())
|
||||||
.spawn(move || actor.run(receiver))
|
.spawn(move || actor.run(receiver))
|
||||||
.expect("failed to spawn thread");
|
.expect("failed to spawn thread");
|
||||||
|
|
Loading…
Add table
Add a link
Reference in a new issue