Monotonic rewrite (#874)

* Rework timer_queue and monotonic architecture

Goals:
 * make Monotonic purely internal
 * make Monotonic purely tick passed, no fugit involved
 * create a wrapper struct in the user's code via a macro that then
   converts the "now" from the tick based monotonic to a fugit based
   timestamp

We need to proxy the delay functions of the timer queue anyway,
so we could simply perform the conversion in those proxy functions.

* Update cargo.lock

* Update readme of rtic-time

* CI: ESP32: Redact esp_image: Too volatile

* Fixup: Changelog double entry rebase mistake

---------

Co-authored-by: Henrik Tjäder <henrik@tjaders.com>
This commit is contained in:
Finomnis 2024-04-11 00:00:38 +02:00 committed by GitHub
parent e4cc5fd17b
commit 8c23e178f3
No known key found for this signature in database
GPG key ID: B5690EEEBB952194
54 changed files with 2637 additions and 1676 deletions

View file

@ -5,285 +5,60 @@
#![no_std]
#![deny(missing_docs)]
#![allow(incomplete_features)]
use core::future::{poll_fn, Future};
use core::pin::Pin;
use core::sync::atomic::{AtomicBool, AtomicUsize, Ordering};
use core::task::{Poll, Waker};
use futures_util::{
future::{select, Either},
pin_mut,
};
use linked_list::{Link, LinkedList};
pub use monotonic::Monotonic;
use rtic_common::dropper::OnDrop;
#![allow(async_fn_in_trait)]
pub mod half_period_counter;
mod linked_list;
mod monotonic;
/// Holds a waker and at which time instant this waker shall be awoken.
struct WaitingWaker<Mono: Monotonic> {
waker: Waker,
release_at: Mono::Instant,
was_popped: AtomicBool,
}
impl<Mono: Monotonic> Clone for WaitingWaker<Mono> {
fn clone(&self) -> Self {
Self {
waker: self.waker.clone(),
release_at: self.release_at,
was_popped: AtomicBool::new(self.was_popped.load(Ordering::Relaxed)),
}
}
}
impl<Mono: Monotonic> PartialEq for WaitingWaker<Mono> {
fn eq(&self, other: &Self) -> bool {
self.release_at == other.release_at
}
}
impl<Mono: Monotonic> PartialOrd for WaitingWaker<Mono> {
fn partial_cmp(&self, other: &Self) -> Option<core::cmp::Ordering> {
self.release_at.partial_cmp(&other.release_at)
}
}
/// A generic timer queue for async executors.
///
/// # Blocking
///
/// The internal priority queue uses global critical sections to manage access. This means that
/// `await`ing a delay will cause a lock of the entire system for O(n) time. In practice the lock
/// duration is ~10 clock cycles per element in the queue.
///
/// # Safety
///
/// This timer queue is based on an intrusive linked list, and by extension the links are strored
/// on the async stacks of callers. The links are deallocated on `drop` or when the wait is
/// complete.
///
/// Do not call `mem::forget` on an awaited future, or there will be dragons!
pub struct TimerQueue<Mono: Monotonic> {
queue: LinkedList<WaitingWaker<Mono>>,
initialized: AtomicBool,
}
pub mod monotonic;
pub mod timer_queue;
/// This indicates that there was a timeout.
pub struct TimeoutError;
/// This is needed to make the async closure in `delay_until` accept that we "share"
/// the link possible between threads.
struct LinkPtr<Mono: Monotonic>(*mut Option<linked_list::Link<WaitingWaker<Mono>>>);
/// Re-export for macros
pub use embedded_hal;
/// Re-export for macros
pub use embedded_hal_async;
impl<Mono: Monotonic> Clone for LinkPtr<Mono> {
fn clone(&self) -> Self {
LinkPtr(self.0)
}
}
impl<Mono: Monotonic> LinkPtr<Mono> {
/// This will dereference the pointer stored within and give out an `&mut`.
unsafe fn get(&mut self) -> &mut Option<linked_list::Link<WaitingWaker<Mono>>> {
&mut *self.0
}
}
unsafe impl<Mono: Monotonic> Send for LinkPtr<Mono> {}
unsafe impl<Mono: Monotonic> Sync for LinkPtr<Mono> {}
impl<Mono: Monotonic> TimerQueue<Mono> {
/// Make a new queue.
pub const fn new() -> Self {
Self {
queue: LinkedList::new(),
initialized: AtomicBool::new(false),
}
}
/// Forwards the `Monotonic::now()` method.
#[inline(always)]
pub fn now(&self) -> Mono::Instant {
Mono::now()
}
/// Takes the initialized monotonic to initialize the TimerQueue.
pub fn initialize(&self, monotonic: Mono) {
self.initialized.store(true, Ordering::SeqCst);
// Don't run drop on `Mono`
core::mem::forget(monotonic);
}
/// Call this in the interrupt handler of the hardware timer supporting the `Monotonic`
/// # A monotonic clock / counter definition.
///
/// ## Correctness
///
/// The trait enforces that proper time-math is implemented between `Instant` and `Duration`. This
/// is a requirement on the time library that the user chooses to use.
pub trait Monotonic {
/// The type for instant, defining an instant in time.
///
/// # Safety
/// **Note:** In all APIs in RTIC that use instants from this monotonic, this type will be used.
type Instant: Ord
+ Copy
+ core::ops::Add<Self::Duration, Output = Self::Instant>
+ core::ops::Sub<Self::Duration, Output = Self::Instant>
+ core::ops::Sub<Self::Instant, Output = Self::Duration>;
/// The type for duration, defining a duration of time.
///
/// It's always safe to call, but it must only be called from the interrupt of the
/// monotonic timer for correct operation.
pub unsafe fn on_monotonic_interrupt(&self) {
Mono::clear_compare_flag();
Mono::on_interrupt();
/// **Note:** In all APIs in RTIC that use duration from this monotonic, this type will be used.
type Duration: Copy;
loop {
let mut release_at = None;
let head = self.queue.pop_if(|head| {
release_at = Some(head.release_at);
/// Get the current time.
fn now() -> Self::Instant;
let should_pop = Mono::now() >= head.release_at;
head.was_popped.store(should_pop, Ordering::Relaxed);
should_pop
});
match (head, release_at) {
(Some(link), _) => {
link.waker.wake();
}
(None, Some(instant)) => {
Mono::enable_timer();
Mono::set_compare(instant);
if Mono::now() >= instant {
// The time for the next instant passed while handling it,
// continue dequeueing
continue;
}
break;
}
(None, None) => {
// Queue is empty
Mono::disable_timer();
break;
}
}
}
}
/// Timeout at a specific time.
pub async fn timeout_at<F: Future>(
&self,
instant: Mono::Instant,
future: F,
) -> Result<F::Output, TimeoutError> {
let delay = self.delay_until(instant);
pin_mut!(future);
pin_mut!(delay);
match select(future, delay).await {
Either::Left((r, _)) => Ok(r),
Either::Right(_) => Err(TimeoutError),
}
}
/// Timeout after at least a specific duration.
#[inline]
pub async fn timeout_after<F: Future>(
&self,
duration: Mono::Duration,
future: F,
) -> Result<F::Output, TimeoutError> {
let now = Mono::now();
let mut timeout = now + duration;
if now != timeout {
timeout = timeout + Mono::TICK_PERIOD;
}
// Wait for one period longer, because by definition timers have an uncertainty
// of one period, so waiting for 'at least' needs to compensate for that.
self.timeout_at(timeout, future).await
}
/// Delay for at least some duration of time.
#[inline]
pub async fn delay(&self, duration: Mono::Duration) {
let now = Mono::now();
let mut timeout = now + duration;
if now != timeout {
timeout = timeout + Mono::TICK_PERIOD;
}
// Wait for one period longer, because by definition timers have an uncertainty
// of one period, so waiting for 'at least' needs to compensate for that.
self.delay_until(timeout).await;
}
/// Delay for some duration of time.
async fn delay(duration: Self::Duration);
/// Delay to some specific time instant.
pub async fn delay_until(&self, instant: Mono::Instant) {
if !self.initialized.load(Ordering::Relaxed) {
panic!(
"The timer queue is not initialized with a monotonic, you need to run `initialize`"
);
}
async fn delay_until(instant: Self::Instant);
let mut link_ptr: Option<linked_list::Link<WaitingWaker<Mono>>> = None;
/// Timeout at a specific time.
async fn timeout_at<F: core::future::Future>(
instant: Self::Instant,
future: F,
) -> Result<F::Output, TimeoutError>;
// Make this future `Drop`-safe
// SAFETY(link_ptr): Shadow the original definition of `link_ptr` so we can't abuse it.
let mut link_ptr =
LinkPtr(&mut link_ptr as *mut Option<linked_list::Link<WaitingWaker<Mono>>>);
let mut link_ptr2 = link_ptr.clone();
let queue = &self.queue;
let marker = &AtomicUsize::new(0);
let dropper = OnDrop::new(|| {
queue.delete(marker.load(Ordering::Relaxed));
});
poll_fn(|cx| {
if Mono::now() >= instant {
return Poll::Ready(());
}
// SAFETY: This pointer is only dereferenced here and on drop of the future
// which happens outside this `poll_fn`'s stack frame, so this mutable access cannot
// happen at the same time as `dropper` runs.
let link = unsafe { link_ptr2.get() };
if link.is_none() {
let link_ref = link.insert(Link::new(WaitingWaker {
waker: cx.waker().clone(),
release_at: instant,
was_popped: AtomicBool::new(false),
}));
// SAFETY(new_unchecked): The address to the link is stable as it is defined
//outside this stack frame.
// SAFETY(insert): `link_ref` lifetime comes from `link_ptr` that is shadowed, and
// we make sure in `dropper` that the link is removed from the queue before
// dropping `link_ptr` AND `dropper` makes sure that the shadowed `link_ptr` lives
// until the end of the stack frame.
let (head_updated, addr) = unsafe { queue.insert(Pin::new_unchecked(link_ref)) };
marker.store(addr, Ordering::Relaxed);
if head_updated {
// Pend the monotonic handler if the queue head was updated.
Mono::pend_interrupt()
}
}
Poll::Pending
})
.await;
// SAFETY: We only run this and dereference the pointer if we have
// exited the `poll_fn` below in the `drop(dropper)` call. The other dereference
// of this pointer is in the `poll_fn`.
if let Some(link) = unsafe { link_ptr.get() } {
if link.val.was_popped.load(Ordering::Relaxed) {
// If it was popped from the queue there is no need to run delete
dropper.defuse();
}
} else {
// Make sure that our link is deleted from the list before we drop this stack
drop(dropper);
}
}
/// Timeout after a specific duration.
async fn timeout_after<F: core::future::Future>(
duration: Self::Duration,
future: F,
) -> Result<F::Output, TimeoutError>;
}

View file

@ -1,236 +1,8 @@
//! A monotonic clock / counter definition.
//! Structs and traits surrounding the [`Monotonic`](crate::Monotonic) trait.
/// # A monotonic clock / counter definition.
///
/// ## Correctness
///
/// The trait enforces that proper time-math is implemented between `Instant` and `Duration`. This
/// is a requirement on the time library that the user chooses to use.
pub trait Monotonic {
/// The time at time zero.
const ZERO: Self::Instant;
pub use timer_queue_based_monotonic::{
TimerQueueBasedDuration, TimerQueueBasedInstant, TimerQueueBasedMonotonic,
};
/// The duration between two timer ticks.
const TICK_PERIOD: Self::Duration;
/// The type for instant, defining an instant in time.
///
/// **Note:** In all APIs in RTIC that use instants from this monotonic, this type will be used.
type Instant: Ord
+ Copy
+ core::ops::Add<Self::Duration, Output = Self::Instant>
+ core::ops::Sub<Self::Duration, Output = Self::Instant>
+ core::ops::Sub<Self::Instant, Output = Self::Duration>;
/// The type for duration, defining an duration of time.
///
/// **Note:** In all APIs in RTIC that use duration from this monotonic, this type will be used.
type Duration;
/// Get the current time.
fn now() -> Self::Instant;
/// Set the compare value of the timer interrupt.
///
/// **Note:** This method does not need to handle race conditions of the monotonic, the timer
/// queue in RTIC checks this.
fn set_compare(instant: Self::Instant);
/// This method used to be required by an errata workaround
/// for the nrf52 family, but it has been disabled as the
/// workaround was erroneous.
#[deprecated(
since = "1.2.0",
note = "this method is erroneous and has been disabled"
)]
fn should_dequeue_check(_: Self::Instant) -> bool {
panic!("This method should not be used as it is erroneous.")
}
/// Clear the compare interrupt flag.
fn clear_compare_flag();
/// Pend the timer's interrupt.
fn pend_interrupt();
/// Optional. Runs on interrupt before any timer queue handling.
fn on_interrupt() {}
/// Optional. This is used to save power, this is called when the timer queue is not empty.
///
/// Enabling and disabling the monotonic needs to propagate to `now` so that an instant
/// based of `now()` is still valid.
///
/// NOTE: This may be called more than once.
fn enable_timer() {}
/// Optional. This is used to save power, this is called when the timer queue is empty.
///
/// Enabling and disabling the monotonic needs to propagate to `now` so that an instant
/// based of `now()` is still valid.
///
/// NOTE: This may be called more than once.
fn disable_timer() {}
}
/// Creates impl blocks for [`embedded_hal::delay::DelayNs`][DelayNs],
/// based on [`fugit::ExtU64Ceil`][ExtU64Ceil].
///
/// [DelayNs]: https://docs.rs/embedded-hal/latest/embedded_hal/delay/trait.DelayNs.html
/// [ExtU64Ceil]: https://docs.rs/fugit/latest/fugit/trait.ExtU64Ceil.html
#[macro_export]
macro_rules! embedded_hal_delay_impl_fugit64 {
($t:ty) => {
impl ::embedded_hal::delay::DelayNs for $t {
fn delay_ns(&mut self, ns: u32) {
use ::fugit::ExtU64Ceil;
let now = Self::now();
let mut done = now + u64::from(ns).nanos_at_least();
if now != done {
// Compensate for sub-tick uncertainty
done += Self::TICK_PERIOD;
}
while Self::now() < done {}
}
fn delay_us(&mut self, us: u32) {
use ::fugit::ExtU64Ceil;
let now = Self::now();
let mut done = now + u64::from(us).micros_at_least();
if now != done {
// Compensate for sub-tick uncertainty
done += Self::TICK_PERIOD;
}
while Self::now() < done {}
}
fn delay_ms(&mut self, ms: u32) {
use ::fugit::ExtU64Ceil;
let now = Self::now();
let mut done = now + u64::from(ms).millis_at_least();
if now != done {
// Compensate for sub-tick uncertainty
done += Self::TICK_PERIOD;
}
while Self::now() < done {}
}
}
};
}
/// Creates impl blocks for [`embedded_hal_async::delay::DelayNs`][DelayNs],
/// based on [`fugit::ExtU64Ceil`][ExtU64Ceil].
///
/// [DelayNs]: https://docs.rs/embedded-hal-async/latest/embedded_hal_async/delay/trait.DelayNs.html
/// [ExtU64Ceil]: https://docs.rs/fugit/latest/fugit/trait.ExtU64Ceil.html
#[macro_export]
macro_rules! embedded_hal_async_delay_impl_fugit64 {
($t:ty) => {
impl ::embedded_hal_async::delay::DelayNs for $t {
#[inline]
async fn delay_ns(&mut self, ns: u32) {
use ::fugit::ExtU64Ceil;
Self::delay(u64::from(ns).nanos_at_least()).await;
}
#[inline]
async fn delay_us(&mut self, us: u32) {
use ::fugit::ExtU64Ceil;
Self::delay(u64::from(us).micros_at_least()).await;
}
#[inline]
async fn delay_ms(&mut self, ms: u32) {
use ::fugit::ExtU64Ceil;
Self::delay(u64::from(ms).millis_at_least()).await;
}
}
};
}
/// Creates impl blocks for [`embedded_hal::delay::DelayNs`][DelayNs],
/// based on [`fugit::ExtU32Ceil`][ExtU32Ceil].
///
/// [DelayNs]: https://docs.rs/embedded-hal/latest/embedded_hal/delay/trait.DelayNs.html
/// [ExtU32Ceil]: https://docs.rs/fugit/latest/fugit/trait.ExtU32Ceil.html
#[macro_export]
macro_rules! embedded_hal_delay_impl_fugit32 {
($t:ty) => {
impl ::embedded_hal::delay::DelayNs for $t {
fn delay_ns(&mut self, ns: u32) {
use ::fugit::ExtU32Ceil;
let now = Self::now();
let mut done = now + ns.nanos_at_least();
if now != done {
// Compensate for sub-tick uncertainty
done += Self::TICK_PERIOD;
}
while Self::now() < done {}
}
fn delay_us(&mut self, us: u32) {
use ::fugit::ExtU32Ceil;
let now = Self::now();
let mut done = now + us.micros_at_least();
if now != done {
// Compensate for sub-tick uncertainty
done += Self::TICK_PERIOD;
}
while Self::now() < done {}
}
fn delay_ms(&mut self, ms: u32) {
use ::fugit::ExtU32Ceil;
let now = Self::now();
let mut done = now + ms.millis_at_least();
if now != done {
// Compensate for sub-tick uncertainty
done += Self::TICK_PERIOD;
}
while Self::now() < done {}
}
}
};
}
/// Creates impl blocks for [`embedded_hal_async::delay::DelayNs`][DelayNs],
/// based on [`fugit::ExtU32Ceil`][ExtU32Ceil].
///
/// [DelayNs]: https://docs.rs/embedded-hal-async/latest/embedded_hal_async/delay/trait.DelayNs.html
/// [ExtU32Ceil]: https://docs.rs/fugit/latest/fugit/trait.ExtU32Ceil.html
#[macro_export]
macro_rules! embedded_hal_async_delay_impl_fugit32 {
($t:ty) => {
impl ::embedded_hal_async::delay::DelayNs for $t {
#[inline]
async fn delay_ns(&mut self, ns: u32) {
use ::fugit::ExtU32Ceil;
Self::delay(ns.nanos_at_least()).await;
}
#[inline]
async fn delay_us(&mut self, us: u32) {
use ::fugit::ExtU32Ceil;
Self::delay(us.micros_at_least()).await;
}
#[inline]
async fn delay_ms(&mut self, ms: u32) {
use ::fugit::ExtU32Ceil;
Self::delay(ms.millis_at_least()).await;
}
}
};
}
mod embedded_hal_macros;
mod timer_queue_based_monotonic;

View file

@ -0,0 +1,77 @@
//! Macros that implement embedded-hal traits for Monotonics
/// Implements [`embedded_hal::delay::DelayNs`] for a given monotonic.
#[macro_export]
macro_rules! impl_embedded_hal_delay_fugit {
($t:ty) => {
impl $crate::embedded_hal::delay::DelayNs for $t {
fn delay_ns(&mut self, ns: u32) {
let now = <Self as $crate::Monotonic>::now();
let mut done =
now + <Self as $crate::Monotonic>::Duration::nanos_at_least(ns.into());
if now != done {
// Compensate for sub-tick uncertainty
done = done + <Self as $crate::Monotonic>::Duration::from_ticks(1);
}
while <Self as $crate::Monotonic>::now() < done {}
}
fn delay_us(&mut self, us: u32) {
let now = <Self as $crate::Monotonic>::now();
let mut done =
now + <Self as $crate::Monotonic>::Duration::micros_at_least(us.into());
if now != done {
// Compensate for sub-tick uncertainty
done = done + <Self as $crate::Monotonic>::Duration::from_ticks(1);
}
while <Self as $crate::Monotonic>::now() < done {}
}
fn delay_ms(&mut self, ms: u32) {
let now = <Self as $crate::Monotonic>::now();
let mut done =
now + <Self as $crate::Monotonic>::Duration::millis_at_least(ms.into());
if now != done {
// Compensate for sub-tick uncertainty
done = done + <Self as $crate::Monotonic>::Duration::from_ticks(1);
}
while <Self as $crate::Monotonic>::now() < done {}
}
}
};
}
/// Implements [`embedded_hal_async::delay::DelayNs`] for a given monotonic.
#[macro_export]
macro_rules! impl_embedded_hal_async_delay_fugit {
($t:ty) => {
impl $crate::embedded_hal_async::delay::DelayNs for $t {
#[inline]
async fn delay_ns(&mut self, ns: u32) {
<Self as $crate::Monotonic>::delay(
<Self as $crate::Monotonic>::Duration::nanos_at_least(ns.into()),
)
.await;
}
#[inline]
async fn delay_us(&mut self, us: u32) {
<Self as $crate::Monotonic>::delay(
<Self as $crate::Monotonic>::Duration::micros_at_least(us.into()),
)
.await;
}
#[inline]
async fn delay_ms(&mut self, ms: u32) {
<Self as $crate::Monotonic>::delay(
<Self as $crate::Monotonic>::Duration::millis_at_least(ms.into()),
)
.await;
}
}
};
}

View file

@ -0,0 +1,113 @@
use crate::{timer_queue::TimerQueueBackend, TimeoutError};
use crate::Monotonic;
/// A [`Monotonic`] that is backed by the [`TimerQueue`](crate::timer_queue::TimerQueue).
pub trait TimerQueueBasedMonotonic {
/// The backend for the timer queue
type Backend: TimerQueueBackend;
/// The type for instant, defining an instant in time.
///
/// **Note:** In all APIs in RTIC that use instants from this monotonic, this type will be used.
type Instant: TimerQueueBasedInstant<Ticks = <Self::Backend as TimerQueueBackend>::Ticks>
+ core::ops::Add<Self::Duration, Output = Self::Instant>
+ core::ops::Sub<Self::Duration, Output = Self::Instant>
+ core::ops::Sub<Self::Instant, Output = Self::Duration>;
/// The type for duration, defining a duration of time.
///
/// **Note:** In all APIs in RTIC that use duration from this monotonic, this type will be used.
type Duration: TimerQueueBasedDuration<Ticks = <Self::Backend as TimerQueueBackend>::Ticks>;
}
impl<T: TimerQueueBasedMonotonic> Monotonic for T {
type Instant = T::Instant;
type Duration = T::Duration;
fn now() -> Self::Instant {
Self::Instant::from_ticks(T::Backend::timer_queue().now())
}
async fn delay(duration: Self::Duration) {
T::Backend::timer_queue().delay(duration.ticks()).await
}
async fn delay_until(instant: Self::Instant) {
T::Backend::timer_queue().delay_until(instant.ticks()).await
}
async fn timeout_at<F: core::future::Future>(
instant: Self::Instant,
future: F,
) -> Result<F::Output, TimeoutError> {
T::Backend::timer_queue()
.timeout_at(instant.ticks(), future)
.await
}
async fn timeout_after<F: core::future::Future>(
duration: Self::Duration,
future: F,
) -> Result<F::Output, TimeoutError> {
T::Backend::timer_queue()
.timeout_after(duration.ticks(), future)
.await
}
}
/// An instant that can be used in [`TimerQueueBasedMonotonic`].
pub trait TimerQueueBasedInstant: Ord + Copy {
/// The internal type of the instant
type Ticks;
/// Convert from ticks to the instant
fn from_ticks(ticks: Self::Ticks) -> Self;
/// Convert the instant to ticks
fn ticks(self) -> Self::Ticks;
}
/// A duration that can be used in [`TimerQueueBasedMonotonic`].
pub trait TimerQueueBasedDuration: Copy {
/// The internal type of the duration
type Ticks;
/// Convert the duration to ticks
fn ticks(self) -> Self::Ticks;
}
impl<const NOM: u32, const DENOM: u32> TimerQueueBasedInstant for fugit::Instant<u64, NOM, DENOM> {
type Ticks = u64;
fn from_ticks(ticks: Self::Ticks) -> Self {
Self::from_ticks(ticks)
}
fn ticks(self) -> Self::Ticks {
Self::ticks(&self)
}
}
impl<const NOM: u32, const DENOM: u32> TimerQueueBasedInstant for fugit::Instant<u32, NOM, DENOM> {
type Ticks = u32;
fn from_ticks(ticks: Self::Ticks) -> Self {
Self::from_ticks(ticks)
}
fn ticks(self) -> Self::Ticks {
Self::ticks(&self)
}
}
impl<const NOM: u32, const DENOM: u32> TimerQueueBasedDuration
for fugit::Duration<u64, NOM, DENOM>
{
type Ticks = u64;
fn ticks(self) -> Self::Ticks {
Self::ticks(&self)
}
}
impl<const NOM: u32, const DENOM: u32> TimerQueueBasedDuration
for fugit::Duration<u32, NOM, DENOM>
{
type Ticks = u32;
fn ticks(self) -> Self::Ticks {
Self::ticks(&self)
}
}

View file

@ -0,0 +1,281 @@
//! A generic timer queue for async executors.
use crate::linked_list::{self, Link, LinkedList};
use crate::TimeoutError;
use core::future::{poll_fn, Future};
use core::pin::Pin;
use core::sync::atomic::{AtomicBool, AtomicUsize, Ordering};
use core::task::{Poll, Waker};
use futures_util::{
future::{select, Either},
pin_mut,
};
use rtic_common::dropper::OnDrop;
mod backend;
mod tick_type;
pub use backend::TimerQueueBackend;
pub use tick_type::TimerQueueTicks;
/// Holds a waker and at which time instant this waker shall be awoken.
struct WaitingWaker<Backend: TimerQueueBackend> {
waker: Waker,
release_at: Backend::Ticks,
was_popped: AtomicBool,
}
impl<Backend: TimerQueueBackend> Clone for WaitingWaker<Backend> {
fn clone(&self) -> Self {
Self {
waker: self.waker.clone(),
release_at: self.release_at,
was_popped: AtomicBool::new(self.was_popped.load(Ordering::Relaxed)),
}
}
}
impl<Backend: TimerQueueBackend> PartialEq for WaitingWaker<Backend> {
fn eq(&self, other: &Self) -> bool {
self.release_at == other.release_at
}
}
impl<Backend: TimerQueueBackend> PartialOrd for WaitingWaker<Backend> {
fn partial_cmp(&self, other: &Self) -> Option<core::cmp::Ordering> {
Some(self.release_at.compare(other.release_at))
}
}
/// A generic timer queue for async executors.
///
/// # Blocking
///
/// The internal priority queue uses global critical sections to manage access. This means that
/// `await`ing a delay will cause a lock of the entire system for O(n) time. In practice the lock
/// duration is ~10 clock cycles per element in the queue.
///
/// # Safety
///
/// This timer queue is based on an intrusive linked list, and by extension the links are stored
/// on the async stacks of callers. The links are deallocated on `drop` or when the wait is
/// complete.
///
/// Do not call `mem::forget` on an awaited future, or there will be dragons!
pub struct TimerQueue<Backend: TimerQueueBackend> {
queue: LinkedList<WaitingWaker<Backend>>,
initialized: AtomicBool,
}
/// This is needed to make the async closure in `delay_until` accept that we "share"
/// the link possible between threads.
struct LinkPtr<Backend: TimerQueueBackend>(*mut Option<linked_list::Link<WaitingWaker<Backend>>>);
impl<Backend: TimerQueueBackend> Clone for LinkPtr<Backend> {
fn clone(&self) -> Self {
LinkPtr(self.0)
}
}
impl<Backend: TimerQueueBackend> LinkPtr<Backend> {
/// This will dereference the pointer stored within and give out an `&mut`.
unsafe fn get(&mut self) -> &mut Option<linked_list::Link<WaitingWaker<Backend>>> {
&mut *self.0
}
}
unsafe impl<Backend: TimerQueueBackend> Send for LinkPtr<Backend> {}
unsafe impl<Backend: TimerQueueBackend> Sync for LinkPtr<Backend> {}
impl<Backend: TimerQueueBackend> TimerQueue<Backend> {
/// Make a new queue.
pub const fn new() -> Self {
Self {
queue: LinkedList::new(),
initialized: AtomicBool::new(false),
}
}
/// Forwards the `Monotonic::now()` method.
#[inline(always)]
pub fn now(&self) -> Backend::Ticks {
Backend::now()
}
/// Takes the initialized monotonic to initialize the TimerQueue.
pub fn initialize(&self, backend: Backend) {
self.initialized.store(true, Ordering::SeqCst);
// Don't run drop on `Mono`
core::mem::forget(backend);
}
/// Call this in the interrupt handler of the hardware timer supporting the `Monotonic`
///
/// # Safety
///
/// It's always safe to call, but it must only be called from the interrupt of the
/// monotonic timer for correct operation.
pub unsafe fn on_monotonic_interrupt(&self) {
Backend::clear_compare_flag();
Backend::on_interrupt();
loop {
let mut release_at = None;
let head = self.queue.pop_if(|head| {
release_at = Some(head.release_at);
let should_pop = Backend::now().is_at_least(head.release_at);
head.was_popped.store(should_pop, Ordering::Relaxed);
should_pop
});
match (head, release_at) {
(Some(link), _) => {
link.waker.wake();
}
(None, Some(instant)) => {
Backend::enable_timer();
Backend::set_compare(instant);
if Backend::now().is_at_least(instant) {
// The time for the next instant passed while handling it,
// continue dequeueing
continue;
}
break;
}
(None, None) => {
// Queue is empty
Backend::disable_timer();
break;
}
}
}
}
/// Timeout at a specific time.
pub async fn timeout_at<F: Future>(
&self,
instant: Backend::Ticks,
future: F,
) -> Result<F::Output, TimeoutError> {
let delay = self.delay_until(instant);
pin_mut!(future);
pin_mut!(delay);
match select(future, delay).await {
Either::Left((r, _)) => Ok(r),
Either::Right(_) => Err(TimeoutError),
}
}
/// Timeout after at least a specific duration.
#[inline]
pub async fn timeout_after<F: Future>(
&self,
duration: Backend::Ticks,
future: F,
) -> Result<F::Output, TimeoutError> {
let now = Backend::now();
let mut timeout = now.wrapping_add(duration);
if now != timeout {
timeout = timeout.wrapping_add(Backend::Ticks::ONE_TICK);
}
// Wait for one period longer, because by definition timers have an uncertainty
// of one period, so waiting for 'at least' needs to compensate for that.
self.timeout_at(timeout, future).await
}
/// Delay for at least some duration of time.
#[inline]
pub async fn delay(&self, duration: Backend::Ticks) {
let now = Backend::now();
let mut timeout = now.wrapping_add(duration);
if now != timeout {
timeout = timeout.wrapping_add(Backend::Ticks::ONE_TICK);
}
// Wait for one period longer, because by definition timers have an uncertainty
// of one period, so waiting for 'at least' needs to compensate for that.
self.delay_until(timeout).await;
}
/// Delay to some specific time instant.
pub async fn delay_until(&self, instant: Backend::Ticks) {
if !self.initialized.load(Ordering::Relaxed) {
panic!(
"The timer queue is not initialized with a monotonic, you need to run `initialize`"
);
}
let mut link_ptr: Option<linked_list::Link<WaitingWaker<Backend>>> = None;
// Make this future `Drop`-safe
// SAFETY(link_ptr): Shadow the original definition of `link_ptr` so we can't abuse it.
let mut link_ptr =
LinkPtr(&mut link_ptr as *mut Option<linked_list::Link<WaitingWaker<Backend>>>);
let mut link_ptr2 = link_ptr.clone();
let queue = &self.queue;
let marker = &AtomicUsize::new(0);
let dropper = OnDrop::new(|| {
queue.delete(marker.load(Ordering::Relaxed));
});
poll_fn(|cx| {
if Backend::now().is_at_least(instant) {
return Poll::Ready(());
}
// SAFETY: This pointer is only dereferenced here and on drop of the future
// which happens outside this `poll_fn`'s stack frame, so this mutable access cannot
// happen at the same time as `dropper` runs.
let link = unsafe { link_ptr2.get() };
if link.is_none() {
let link_ref = link.insert(Link::new(WaitingWaker {
waker: cx.waker().clone(),
release_at: instant,
was_popped: AtomicBool::new(false),
}));
// SAFETY(new_unchecked): The address to the link is stable as it is defined
//outside this stack frame.
// SAFETY(insert): `link_ref` lifetime comes from `link_ptr` that is shadowed, and
// we make sure in `dropper` that the link is removed from the queue before
// dropping `link_ptr` AND `dropper` makes sure that the shadowed `link_ptr` lives
// until the end of the stack frame.
let (head_updated, addr) = unsafe { queue.insert(Pin::new_unchecked(link_ref)) };
marker.store(addr, Ordering::Relaxed);
if head_updated {
// Pend the monotonic handler if the queue head was updated.
Backend::pend_interrupt()
}
}
Poll::Pending
})
.await;
// SAFETY: We only run this and dereference the pointer if we have
// exited the `poll_fn` below in the `drop(dropper)` call. The other dereference
// of this pointer is in the `poll_fn`.
if let Some(link) = unsafe { link_ptr.get() } {
if link.val.was_popped.load(Ordering::Relaxed) {
// If it was popped from the queue there is no need to run delete
dropper.defuse();
}
} else {
// Make sure that our link is deleted from the list before we drop this stack
drop(dropper);
}
}
}

View file

@ -0,0 +1,44 @@
use super::{TimerQueue, TimerQueueTicks};
/// A backend definition for a monotonic clock/counter.
pub trait TimerQueueBackend: 'static + Sized {
/// The type for ticks.
type Ticks: TimerQueueTicks;
/// Get the current time.
fn now() -> Self::Ticks;
/// Set the compare value of the timer interrupt.
///
/// **Note:** This method does not need to handle race conditions of the monotonic, the timer
/// queue in RTIC checks this.
fn set_compare(instant: Self::Ticks);
/// Clear the compare interrupt flag.
fn clear_compare_flag();
/// Pend the timer's interrupt.
fn pend_interrupt();
/// Optional. Runs on interrupt before any timer queue handling.
fn on_interrupt() {}
/// Optional. This is used to save power, this is called when the timer queue is not empty.
///
/// Enabling and disabling the monotonic needs to propagate to `now` so that an instant
/// based of `now()` is still valid.
///
/// NOTE: This may be called more than once.
fn enable_timer() {}
/// Optional. This is used to save power, this is called when the timer queue is empty.
///
/// Enabling and disabling the monotonic needs to propagate to `now` so that an instant
/// based of `now()` is still valid.
///
/// NOTE: This may be called more than once.
fn disable_timer() {}
/// Returns a reference to the underlying timer queue.
fn timer_queue() -> &'static TimerQueue<Self>;
}

View file

@ -0,0 +1,49 @@
use core::cmp;
/// The ticks of a timer.
pub trait TimerQueueTicks: Copy + PartialEq + Eq {
/// Represents a single tick.
const ONE_TICK: Self;
/// Compares to another tick count.
///
/// Takes into account timer wrapping; if the difference is more than
/// half the value range, the result will be flipped.
fn compare(self, other: Self) -> cmp::Ordering;
/// True if `self` is at the same time as `other` or later.
///
/// Takes into account timer wrapping; if the difference is more than
/// half the value range, the result will be negated.
fn is_at_least(self, other: Self) -> bool {
match self.compare(other) {
cmp::Ordering::Less => false,
cmp::Ordering::Equal => true,
cmp::Ordering::Greater => true,
}
}
/// Wrapping addition.
fn wrapping_add(self, other: Self) -> Self;
}
impl TimerQueueTicks for u32 {
const ONE_TICK: Self = 1;
fn compare(self, other: Self) -> cmp::Ordering {
(self.wrapping_sub(other) as i32).cmp(&0)
}
fn wrapping_add(self, other: Self) -> Self {
u32::wrapping_add(self, other)
}
}
impl TimerQueueTicks for u64 {
const ONE_TICK: Self = 1;
fn compare(self, other: Self) -> cmp::Ordering {
(self.wrapping_sub(other) as i64).cmp(&0)
}
fn wrapping_add(self, other: Self) -> Self {
u64::wrapping_add(self, other)
}
}