ts/executor: replace tokio with smol-like implementation

The threadshare executor was based on a modified version of tokio
which implemented the throttling strategy in the BasicScheduler.
Upstream tokio codebase has significantly diverged from what it
was when the throttling strategy was implemented making it hard
to follow. This means that we can hardly get updates from the
upstream project and when we cherry pick fixes, we can't reflect
the state of the project on our fork's version. As a consequence,
tools such as cargo-deny can't check for RUSTSEC fixes in our fork.

The smol ecosystem makes it quite easy to implement and maintain
a custom async executor. This MR imports the smol parts that
need modifications to comply with the threadshare model and implements
a throttling executor in place of the tokio fork.

Networking tokio specific types are replaced with Async wrappers
in the spirit of [smol-rs/async-io]. Note however that the Async
wrappers needed modifications in order to use the per thread
Reactor model. This means that higher level upstream networking
crates such as [async-net] can not be used with our Async
implementation.

Based on the example benchmark with ts-udpsrc, performances seem on par
with what we achieved using the tokio fork.

Fixes https://gitlab.freedesktop.org/gstreamer/gst-plugins-rs/-/issues/118

Related to https://gitlab.freedesktop.org/gstreamer/gst-plugins-rs/-/merge_requests/604
This commit is contained in:
François Laignel 2021-12-14 19:40:27 +01:00 committed by Sebastian Dröge
parent db9c38aa93
commit 6163589ac7
24 changed files with 2638 additions and 551 deletions

View file

@ -9,20 +9,24 @@ edition = "2021"
rust-version = "1.56"
[dependencies]
async-task = "4.0.3"
concurrent-queue = "1.2.2"
futures = { version = "0.3.17", features = ["thread-pool"] }
libc = "0.2"
gio = { git = "https://github.com/gtk-rs/gtk-rs-core" }
gst = { package = "gstreamer", git = "https://gitlab.freedesktop.org/gstreamer/gstreamer-rs" }
gst-net = { package = "gstreamer-net", git = "https://gitlab.freedesktop.org/gstreamer/gstreamer-rs" }
gst-rtp = { package = "gstreamer-rtp", git = "https://gitlab.freedesktop.org/gstreamer/gstreamer-rs" }
once_cell = "1"
pin-project-lite = "0.2.7"
tokio = { git = "https://github.com/fengalin/tokio", tag = "tokio-0.2.13-throttling.1", features = ["io-util", "macros", "rt-core", "sync", "stream", "time", "tcp", "udp", "rt-util"] }
futures = { version = "0.3", features = ["thread-pool"] }
pin-project-lite = "0.2.0"
polling = "2.0.0"
rand = "0.8"
slab = "0.4.2"
socket2 = {features = ["all"], version = "0.4"}
waker-fn = "1.1"
[target.'cfg(windows)'.dependencies]
winapi = { version = "0.3", features = ["winsock2", "processthreadsapi"] }
winapi = { version = "0.3.9", features = ["winsock2", "processthreadsapi"] }
[dev-dependencies]
gst-check = { package = "gstreamer-check", git = "https://gitlab.freedesktop.org/gstreamer/gstreamer-rs" }

View file

@ -190,7 +190,7 @@ fn main() {
if let Some(prev_reset_instant) = prev_reset_instant {
println!(
"{:>5.1} / s / stream",
"{:>6.2} / s / stream",
(count as f32) * throughput_factor
/ ((reset_instant - prev_reset_instant).as_millis() as f32)
);

View file

@ -617,7 +617,7 @@ impl ObjectImpl for AppSrc {
settings.context = value
.get::<Option<String>>()
.expect("type checked upstream")
.unwrap_or_else(|| "".into());
.unwrap_or_else(|| DEFAULT_CONTEXT.into());
}
"context-wait" => {
settings.context_wait = Duration::from_millis(

View file

@ -447,7 +447,7 @@ impl ObjectImpl for InputSelector {
settings.context = value
.get::<Option<String>>()
.expect("type checked upstream")
.unwrap_or_else(|| "".into());
.unwrap_or_else(|| DEFAULT_CONTEXT.into());
}
"context-wait" => {
let mut settings = self.settings.lock().unwrap();

View file

@ -1452,7 +1452,7 @@ impl ObjectImpl for JitterBuffer {
settings.context = value
.get::<Option<String>>()
.expect("type checked upstream")
.unwrap_or_else(|| "".into());
.unwrap_or_else(|| DEFAULT_CONTEXT.into());
}
"context-wait" => {
let mut settings = self.settings.lock().unwrap();

View file

@ -1,30 +1,11 @@
// Copyright (C) 2018 Sebastian Dröge <sebastian@centricular.com>
//
// This library is free software; you can redistribute it and/or
// modify it under the terms of the GNU Library General Public
// License as published by the Free Software Foundation; either
// version 2 of the License, or (at your option) any later version.
//
// This library is distributed in the hope that it will be useful,
// but WITHOUT ANY WARRANTY; without even the implied warranty of
// MERCHANTABILITY or FITNESS FOR A PARTICULAR PURPOSE. See the GNU
// Library General Public License for more details.
//
// You should have received a copy of the GNU Library General Public
// License along with this library; if not, write to the
// Free Software Foundation, Inc., 51 Franklin Street, Suite 500,
// Boston, MA 02110-1335, USA.
// Take a look at the license at the top of the repository in the LICENSE file.
//! A collection of GStreamer plugins which leverage the `threadshare` [`runtime`].
//!
//! [`runtime`]: runtime/index.html
// Needed for `select!` in `Socket::next`
// see https://docs.rs/futures/0.3.1/futures/macro.select.html
#![recursion_limit = "1024"]
pub use tokio;
#[macro_use]
pub mod runtime;

View file

@ -614,7 +614,7 @@ impl ObjectImpl for ProxySink {
settings.proxy_context = value
.get::<Option<String>>()
.expect("type checked upstream")
.unwrap_or_else(|| "".into());
.unwrap_or_else(|| DEFAULT_PROXY_CONTEXT.into());
}
_ => unimplemented!(),
}
@ -1221,7 +1221,7 @@ impl ObjectImpl for ProxySrc {
settings.proxy_context = value
.get::<Option<String>>()
.expect("type checked upstream")
.unwrap_or_else(|| "".into());
.unwrap_or_else(|| DEFAULT_PROXY_CONTEXT.into());
}
_ => unimplemented!(),
}

View file

@ -789,7 +789,7 @@ impl ObjectImpl for Queue {
settings.context = value
.get::<Option<String>>()
.expect("type checked upstream")
.unwrap_or_else(|| "".into());
.unwrap_or_else(|| DEFAULT_CONTEXT.into());
}
"context-wait" => {
settings.context_wait = Duration::from_millis(

View file

@ -0,0 +1,885 @@
// This is based on https://github.com/smol-rs/async-io
// with adaptations by:
//
// Copyright (C) 2021 François Laignel <fengalin@free.fr>
//
// Take a look at the license at the top of the repository in the LICENSE file.
use futures::io::{AsyncRead, AsyncWrite};
use futures::stream::{self, Stream};
use futures::{future, pin_mut, ready};
use std::future::Future;
use std::io::{self, IoSlice, IoSliceMut, Read, Write};
use std::net::{SocketAddr, TcpListener, TcpStream, UdpSocket};
use std::pin::Pin;
use std::sync::Arc;
use std::task::{Context, Poll};
#[cfg(unix)]
use std::{
os::unix::io::{AsRawFd, RawFd},
os::unix::net::{SocketAddr as UnixSocketAddr, UnixDatagram, UnixListener, UnixStream},
path::Path,
};
#[cfg(windows)]
use std::os::windows::io::{AsRawSocket, RawSocket};
use socket2::{Domain, Protocol, SockAddr, Socket, Type};
use super::scheduler::{self, Scheduler};
use super::{Reactor, Readable, ReadableOwned, Source, Writable, WritableOwned};
/// Async adapter for I/O types.
///
/// This type puts an I/O handle into non-blocking mode, registers it in
/// [epoll]/[kqueue]/[event ports]/[wepoll], and then provides an async interface for it.
///
/// [epoll]: https://en.wikipedia.org/wiki/Epoll
/// [kqueue]: https://en.wikipedia.org/wiki/Kqueue
/// [event ports]: https://illumos.org/man/port_create
/// [wepoll]: https://github.com/piscisaureus/wepoll
///
/// # Caveats
///
/// The [`Async`] implementation is specific to the threadshare implementation.
/// Neither [`async-net`] nor [`async-process`] (on Unix) can be used.
///
/// [`async-net`]: https://github.com/smol-rs/async-net
/// [`async-process`]: https://github.com/smol-rs/async-process
///
/// ### Supported types
///
/// [`Async`] supports all networking types, as well as some OS-specific file descriptors like
/// [timerfd] and [inotify].
///
/// However, do not use [`Async`] with types like [`File`][`std::fs::File`],
/// [`Stdin`][`std::io::Stdin`], [`Stdout`][`std::io::Stdout`], or [`Stderr`][`std::io::Stderr`]
/// because all operating systems have issues with them when put in non-blocking mode.
///
/// [timerfd]: https://github.com/smol-rs/async-io/blob/master/examples/linux-timerfd.rs
/// [inotify]: https://github.com/smol-rs/async-io/blob/master/examples/linux-inotify.rs
///
/// ### Concurrent I/O
///
/// Note that [`&Async<T>`][`Async`] implements [`AsyncRead`] and [`AsyncWrite`] if `&T`
/// implements those traits, which means tasks can concurrently read and write using shared
/// references.
///
/// But there is a catch: only one task can read a time, and only one task can write at a time. It
/// is okay to have two tasks where one is reading and the other is writing at the same time, but
/// it is not okay to have two tasks reading at the same time or writing at the same time. If you
/// try to do that, conflicting tasks will just keep waking each other in turn, thus wasting CPU
/// time.
///
/// Besides [`AsyncRead`] and [`AsyncWrite`], this caveat also applies to
/// [`poll_readable()`][`Async::poll_readable()`] and
/// [`poll_writable()`][`Async::poll_writable()`].
///
/// However, any number of tasks can be concurrently calling other methods like
/// [`readable()`][`Async::readable()`] or [`read_with()`][`Async::read_with()`].
///
/// ### Closing
///
/// Closing the write side of [`Async`] with [`close()`][`futures::AsyncWriteExt::close()`]
/// simply flushes. If you want to shutdown a TCP or Unix socket, use
/// [`Shutdown`][`std::net::Shutdown`].
///
#[derive(Debug)]
pub struct Async<T> {
/// A source registered in the reactor.
pub(super) source: Arc<Source>,
/// The inner I/O handle.
io: Option<T>,
// The [`Handle`] on the [`Scheduler`] on which this Async wrapper is registered.
handle: scheduler::HandleWeak,
}
impl<T> Unpin for Async<T> {}
#[cfg(unix)]
impl<T: AsRawFd> Async<T> {
/// Creates an async I/O handle.
///
/// This method will put the handle in non-blocking mode and register it in
/// [epoll]/[kqueue]/[event ports]/[wepoll].
///
/// On Unix systems, the handle must implement `AsRawFd`, while on Windows it must implement
/// `AsRawSocket`.
///
/// [epoll]: https://en.wikipedia.org/wiki/Epoll
/// [kqueue]: https://en.wikipedia.org/wiki/Kqueue
/// [event ports]: https://illumos.org/man/port_create
/// [wepoll]: https://github.com/piscisaureus/wepoll
pub fn new(io: T) -> io::Result<Async<T>> {
let fd = io.as_raw_fd();
// Put the file descriptor in non-blocking mode.
unsafe {
let mut res = libc::fcntl(fd, libc::F_GETFL);
if res != -1 {
res = libc::fcntl(fd, libc::F_SETFL, res | libc::O_NONBLOCK);
}
if res == -1 {
return Err(io::Error::last_os_error());
}
}
let source = Reactor::with_mut(|reactor| reactor.insert_io(fd))?;
Ok(Async {
source,
io: Some(io),
handle: Scheduler::current()
.expect("Attempt to create an Async wrapper outside of a Context")
.downgrade(),
})
}
}
#[cfg(unix)]
impl<T: AsRawFd> AsRawFd for Async<T> {
fn as_raw_fd(&self) -> RawFd {
self.source.raw
}
}
#[cfg(windows)]
impl<T: AsRawSocket> Async<T> {
/// Creates an async I/O handle.
///
/// This method will put the handle in non-blocking mode and register it in
/// [epoll]/[kqueue]/[event ports]/[wepoll].
///
/// On Unix systems, the handle must implement `AsRawFd`, while on Windows it must implement
/// `AsRawSocket`.
///
/// [epoll]: https://en.wikipedia.org/wiki/Epoll
/// [kqueue]: https://en.wikipedia.org/wiki/Kqueue
/// [event ports]: https://illumos.org/man/port_create
/// [wepoll]: https://github.com/piscisaureus/wepoll
pub fn new(io: T) -> io::Result<Async<T>> {
let sock = io.as_raw_socket();
// Put the socket in non-blocking mode.
unsafe {
use winapi::ctypes;
use winapi::um::winsock2;
let mut nonblocking = true as ctypes::c_ulong;
let res = winsock2::ioctlsocket(
sock as winsock2::SOCKET,
winsock2::FIONBIO,
&mut nonblocking,
);
if res != 0 {
return Err(io::Error::last_os_error());
}
}
let source = Reactor::with_mut(|reactor| reactor.insert_io(fd))?;
Ok(Async {
source,
io: Some(io),
handle: Scheduler::current()
.expect("Attempt to create an Async wrapper outside of a Context")
.downgrade(),
})
}
}
#[cfg(windows)]
impl<T: AsRawSocket> AsRawSocket for Async<T> {
fn as_raw_socket(&self) -> RawSocket {
self.source.raw
}
}
impl<T> Async<T> {
/// Gets a reference to the inner I/O handle.
pub fn get_ref(&self) -> &T {
self.io.as_ref().unwrap()
}
/// Gets a mutable reference to the inner I/O handle.
pub fn get_mut(&mut self) -> &mut T {
self.io.as_mut().unwrap()
}
/// Unwraps the inner I/O handle.
pub fn into_inner(mut self) -> io::Result<T> {
let io = self.io.take().unwrap();
Reactor::with_mut(|reactor| reactor.remove_io(&self.source))?;
Ok(io)
}
/// Waits until the I/O handle is readable.
///
/// This method completes when a read operation on this I/O handle wouldn't block.
pub fn readable(&self) -> Readable<'_, T> {
Source::readable(self)
}
/// Waits until the I/O handle is readable.
///
/// This method completes when a read operation on this I/O handle wouldn't block.
pub fn readable_owned(self: Arc<Self>) -> ReadableOwned<T> {
Source::readable_owned(self)
}
/// Waits until the I/O handle is writable.
///
/// This method completes when a write operation on this I/O handle wouldn't block.
pub fn writable(&self) -> Writable<'_, T> {
Source::writable(self)
}
/// Waits until the I/O handle is writable.
///
/// This method completes when a write operation on this I/O handle wouldn't block.
pub fn writable_owned(self: Arc<Self>) -> WritableOwned<T> {
Source::writable_owned(self)
}
/// Polls the I/O handle for readability.
///
/// When this method returns [`Poll::Ready`], that means the OS has delivered an event
/// indicating readability since the last time this task has called the method and received
/// [`Poll::Pending`].
///
/// # Caveats
///
/// Two different tasks should not call this method concurrently. Otherwise, conflicting tasks
/// will just keep waking each other in turn, thus wasting CPU time.
///
/// Note that the [`AsyncRead`] implementation for [`Async`] also uses this method.
pub fn poll_readable(&self, cx: &mut Context<'_>) -> Poll<io::Result<()>> {
self.source.poll_readable(cx)
}
/// Polls the I/O handle for writability.
///
/// When this method returns [`Poll::Ready`], that means the OS has delivered an event
/// indicating writability since the last time this task has called the method and received
/// [`Poll::Pending`].
///
/// # Caveats
///
/// Two different tasks should not call this method concurrently. Otherwise, conflicting tasks
/// will just keep waking each other in turn, thus wasting CPU time.
///
/// Note that the [`AsyncWrite`] implementation for [`Async`] also uses this method.
pub fn poll_writable(&self, cx: &mut Context<'_>) -> Poll<io::Result<()>> {
self.source.poll_writable(cx)
}
/// Performs a read operation asynchronously.
///
/// The I/O handle is registered in the reactor and put in non-blocking mode. This method
/// invokes the `op` closure in a loop until it succeeds or returns an error other than
/// [`io::ErrorKind::WouldBlock`]. In between iterations of the loop, it waits until the OS
/// sends a notification that the I/O handle is readable.
///
/// The closure receives a shared reference to the I/O handle.
pub async fn read_with<R>(&self, op: impl FnMut(&T) -> io::Result<R>) -> io::Result<R> {
let mut op = op;
loop {
match op(self.get_ref()) {
Err(err) if err.kind() == io::ErrorKind::WouldBlock => {}
res => return res,
}
optimistic(self.readable()).await?;
}
}
/// Performs a read operation asynchronously.
///
/// The I/O handle is registered in the reactor and put in non-blocking mode. This method
/// invokes the `op` closure in a loop until it succeeds or returns an error other than
/// [`io::ErrorKind::WouldBlock`]. In between iterations of the loop, it waits until the OS
/// sends a notification that the I/O handle is readable.
///
/// The closure receives a mutable reference to the I/O handle.
pub async fn read_with_mut<R>(
&mut self,
op: impl FnMut(&mut T) -> io::Result<R>,
) -> io::Result<R> {
let mut op = op;
loop {
match op(self.get_mut()) {
Err(err) if err.kind() == io::ErrorKind::WouldBlock => {}
res => return res,
}
optimistic(self.readable()).await?;
}
}
/// Performs a write operation asynchronously.
///
/// The I/O handle is registered in the reactor and put in non-blocking mode. This method
/// invokes the `op` closure in a loop until it succeeds or returns an error other than
/// [`io::ErrorKind::WouldBlock`]. In between iterations of the loop, it waits until the OS
/// sends a notification that the I/O handle is writable.
///
/// The closure receives a shared reference to the I/O handle.
pub async fn write_with<R>(&self, op: impl FnMut(&T) -> io::Result<R>) -> io::Result<R> {
let mut op = op;
loop {
match op(self.get_ref()) {
Err(err) if err.kind() == io::ErrorKind::WouldBlock => {}
res => return res,
}
optimistic(self.writable()).await?;
}
}
/// Performs a write operation asynchronously.
///
/// The I/O handle is registered in the reactor and put in non-blocking mode. This method
/// invokes the `op` closure in a loop until it succeeds or returns an error other than
/// [`io::ErrorKind::WouldBlock`]. In between iterations of the loop, it waits until the OS
/// sends a notification that the I/O handle is writable.
///
/// The closure receives a mutable reference to the I/O handle.
pub async fn write_with_mut<R>(
&mut self,
op: impl FnMut(&mut T) -> io::Result<R>,
) -> io::Result<R> {
let mut op = op;
loop {
match op(self.get_mut()) {
Err(err) if err.kind() == io::ErrorKind::WouldBlock => {}
res => return res,
}
optimistic(self.writable()).await?;
}
}
}
impl<T> AsRef<T> for Async<T> {
fn as_ref(&self) -> &T {
self.get_ref()
}
}
impl<T> AsMut<T> for Async<T> {
fn as_mut(&mut self) -> &mut T {
self.get_mut()
}
}
impl<T> Drop for Async<T> {
fn drop(&mut self) {
if let Some(io) = self.io.take() {
// Drop the I/O handle to close it.
drop(io);
if let Some(handle) = self.handle.upgrade() {
handle.remove_soure(Arc::clone(&self.source));
}
}
}
}
impl<T: Read> AsyncRead for Async<T> {
fn poll_read(
mut self: Pin<&mut Self>,
cx: &mut Context<'_>,
buf: &mut [u8],
) -> Poll<io::Result<usize>> {
loop {
match (&mut *self).get_mut().read(buf) {
Err(err) if err.kind() == io::ErrorKind::WouldBlock => {}
res => return Poll::Ready(res),
}
ready!(self.poll_readable(cx))?;
}
}
fn poll_read_vectored(
mut self: Pin<&mut Self>,
cx: &mut Context<'_>,
bufs: &mut [IoSliceMut<'_>],
) -> Poll<io::Result<usize>> {
loop {
match (&mut *self).get_mut().read_vectored(bufs) {
Err(err) if err.kind() == io::ErrorKind::WouldBlock => {}
res => return Poll::Ready(res),
}
ready!(self.poll_readable(cx))?;
}
}
}
impl<T> AsyncRead for &Async<T>
where
for<'a> &'a T: Read,
{
fn poll_read(
self: Pin<&mut Self>,
cx: &mut Context<'_>,
buf: &mut [u8],
) -> Poll<io::Result<usize>> {
loop {
match (*self).get_ref().read(buf) {
Err(err) if err.kind() == io::ErrorKind::WouldBlock => {}
res => return Poll::Ready(res),
}
ready!(self.poll_readable(cx))?;
}
}
fn poll_read_vectored(
self: Pin<&mut Self>,
cx: &mut Context<'_>,
bufs: &mut [IoSliceMut<'_>],
) -> Poll<io::Result<usize>> {
loop {
match (*self).get_ref().read_vectored(bufs) {
Err(err) if err.kind() == io::ErrorKind::WouldBlock => {}
res => return Poll::Ready(res),
}
ready!(self.poll_readable(cx))?;
}
}
}
impl<T: Write> AsyncWrite for Async<T> {
fn poll_write(
mut self: Pin<&mut Self>,
cx: &mut Context<'_>,
buf: &[u8],
) -> Poll<io::Result<usize>> {
loop {
match (&mut *self).get_mut().write(buf) {
Err(err) if err.kind() == io::ErrorKind::WouldBlock => {}
res => return Poll::Ready(res),
}
ready!(self.poll_writable(cx))?;
}
}
fn poll_write_vectored(
mut self: Pin<&mut Self>,
cx: &mut Context<'_>,
bufs: &[IoSlice<'_>],
) -> Poll<io::Result<usize>> {
loop {
match (&mut *self).get_mut().write_vectored(bufs) {
Err(err) if err.kind() == io::ErrorKind::WouldBlock => {}
res => return Poll::Ready(res),
}
ready!(self.poll_writable(cx))?;
}
}
fn poll_flush(mut self: Pin<&mut Self>, cx: &mut Context<'_>) -> Poll<io::Result<()>> {
loop {
match (&mut *self).get_mut().flush() {
Err(err) if err.kind() == io::ErrorKind::WouldBlock => {}
res => return Poll::Ready(res),
}
ready!(self.poll_writable(cx))?;
}
}
fn poll_close(self: Pin<&mut Self>, cx: &mut Context<'_>) -> Poll<io::Result<()>> {
self.poll_flush(cx)
}
}
impl<T> AsyncWrite for &Async<T>
where
for<'a> &'a T: Write,
{
fn poll_write(
self: Pin<&mut Self>,
cx: &mut Context<'_>,
buf: &[u8],
) -> Poll<io::Result<usize>> {
loop {
match (*self).get_ref().write(buf) {
Err(err) if err.kind() == io::ErrorKind::WouldBlock => {}
res => return Poll::Ready(res),
}
ready!(self.poll_writable(cx))?;
}
}
fn poll_write_vectored(
self: Pin<&mut Self>,
cx: &mut Context<'_>,
bufs: &[IoSlice<'_>],
) -> Poll<io::Result<usize>> {
loop {
match (*self).get_ref().write_vectored(bufs) {
Err(err) if err.kind() == io::ErrorKind::WouldBlock => {}
res => return Poll::Ready(res),
}
ready!(self.poll_writable(cx))?;
}
}
fn poll_flush(self: Pin<&mut Self>, cx: &mut Context<'_>) -> Poll<io::Result<()>> {
loop {
match (*self).get_ref().flush() {
Err(err) if err.kind() == io::ErrorKind::WouldBlock => {}
res => return Poll::Ready(res),
}
ready!(self.poll_writable(cx))?;
}
}
fn poll_close(self: Pin<&mut Self>, cx: &mut Context<'_>) -> Poll<io::Result<()>> {
self.poll_flush(cx)
}
}
impl Async<TcpListener> {
/// Creates a TCP listener bound to the specified address.
///
/// Binding with port number 0 will request an available port from the OS.
pub fn bind<A: Into<SocketAddr>>(addr: A) -> io::Result<Async<TcpListener>> {
let addr = addr.into();
Async::new(TcpListener::bind(addr)?)
}
/// Accepts a new incoming TCP connection.
///
/// When a connection is established, it will be returned as a TCP stream together with its
/// remote address.
pub async fn accept(&self) -> io::Result<(Async<TcpStream>, SocketAddr)> {
let (stream, addr) = self.read_with(|io| io.accept()).await?;
Ok((Async::new(stream)?, addr))
}
/// Returns a stream of incoming TCP connections.
///
/// The stream is infinite, i.e. it never stops with a [`None`].
pub fn incoming(&self) -> impl Stream<Item = io::Result<Async<TcpStream>>> + Send + '_ {
stream::unfold(self, |listener| async move {
let res = listener.accept().await.map(|(stream, _)| stream);
Some((res, listener))
})
}
}
impl TryFrom<std::net::TcpListener> for Async<std::net::TcpListener> {
type Error = io::Error;
fn try_from(listener: std::net::TcpListener) -> io::Result<Self> {
Async::new(listener)
}
}
impl Async<TcpStream> {
/// Creates a TCP connection to the specified address.
pub async fn connect<A: Into<SocketAddr>>(addr: A) -> io::Result<Async<TcpStream>> {
// Begin async connect.
let addr = addr.into();
let domain = Domain::for_address(addr);
let socket = connect(addr.into(), domain, Some(Protocol::TCP))?;
let stream = Async::new(TcpStream::from(socket))?;
// The stream becomes writable when connected.
stream.writable().await?;
// Check if there was an error while connecting.
match stream.get_ref().take_error()? {
None => Ok(stream),
Some(err) => Err(err),
}
}
/// Reads data from the stream without removing it from the buffer.
///
/// Returns the number of bytes read. Successive calls of this method read the same data.
pub async fn peek(&self, buf: &mut [u8]) -> io::Result<usize> {
self.read_with(|io| io.peek(buf)).await
}
}
impl TryFrom<std::net::TcpStream> for Async<std::net::TcpStream> {
type Error = io::Error;
fn try_from(stream: std::net::TcpStream) -> io::Result<Self> {
Async::new(stream)
}
}
impl Async<UdpSocket> {
/// Creates a UDP socket bound to the specified address.
///
/// Binding with port number 0 will request an available port from the OS.
pub fn bind<A: Into<SocketAddr>>(addr: A) -> io::Result<Async<UdpSocket>> {
let addr = addr.into();
Async::new(UdpSocket::bind(addr)?)
}
/// Receives a single datagram message.
///
/// Returns the number of bytes read and the address the message came from.
///
/// This method must be called with a valid byte slice of sufficient size to hold the message.
/// If the message is too long to fit, excess bytes may get discarded.
pub async fn recv_from(&self, buf: &mut [u8]) -> io::Result<(usize, SocketAddr)> {
self.read_with(|io| io.recv_from(buf)).await
}
/// Receives a single datagram message without removing it from the queue.
///
/// Returns the number of bytes read and the address the message came from.
///
/// This method must be called with a valid byte slice of sufficient size to hold the message.
/// If the message is too long to fit, excess bytes may get discarded.
pub async fn peek_from(&self, buf: &mut [u8]) -> io::Result<(usize, SocketAddr)> {
self.read_with(|io| io.peek_from(buf)).await
}
/// Sends data to the specified address.
///
/// Returns the number of bytes writen.
pub async fn send_to<A: Into<SocketAddr>>(&self, buf: &[u8], addr: A) -> io::Result<usize> {
let addr = addr.into();
self.write_with(|io| io.send_to(buf, addr)).await
}
/// Receives a single datagram message from the connected peer.
///
/// Returns the number of bytes read.
///
/// This method must be called with a valid byte slice of sufficient size to hold the message.
/// If the message is too long to fit, excess bytes may get discarded.
///
/// The [`connect`][`UdpSocket::connect()`] method connects this socket to a remote address.
/// This method will fail if the socket is not connected.
pub async fn recv(&self, buf: &mut [u8]) -> io::Result<usize> {
self.read_with(|io| io.recv(buf)).await
}
/// Receives a single datagram message from the connected peer without removing it from the
/// queue.
///
/// Returns the number of bytes read and the address the message came from.
///
/// This method must be called with a valid byte slice of sufficient size to hold the message.
/// If the message is too long to fit, excess bytes may get discarded.
///
/// The [`connect`][`UdpSocket::connect()`] method connects this socket to a remote address.
/// This method will fail if the socket is not connected.
pub async fn peek(&self, buf: &mut [u8]) -> io::Result<usize> {
self.read_with(|io| io.peek(buf)).await
}
/// Sends data to the connected peer.
///
/// Returns the number of bytes written.
///
/// The [`connect`][`UdpSocket::connect()`] method connects this socket to a remote address.
/// This method will fail if the socket is not connected.
pub async fn send(&self, buf: &[u8]) -> io::Result<usize> {
self.write_with(|io| io.send(buf)).await
}
}
impl TryFrom<std::net::UdpSocket> for Async<std::net::UdpSocket> {
type Error = io::Error;
fn try_from(socket: std::net::UdpSocket) -> io::Result<Self> {
Async::new(socket)
}
}
impl TryFrom<socket2::Socket> for Async<std::net::UdpSocket> {
type Error = io::Error;
fn try_from(socket: socket2::Socket) -> io::Result<Self> {
Async::new(std::net::UdpSocket::from(socket))
}
}
#[cfg(unix)]
impl Async<UnixListener> {
/// Creates a UDS listener bound to the specified path.
pub fn bind<P: AsRef<Path>>(path: P) -> io::Result<Async<UnixListener>> {
let path = path.as_ref().to_owned();
Async::new(UnixListener::bind(path)?)
}
/// Accepts a new incoming UDS stream connection.
pub async fn accept(&self) -> io::Result<(Async<UnixStream>, UnixSocketAddr)> {
let (stream, addr) = self.read_with(|io| io.accept()).await?;
Ok((Async::new(stream)?, addr))
}
/// Returns a stream of incoming UDS connections.
///
/// The stream is infinite, i.e. it never stops with a [`None`] item.
pub fn incoming(&self) -> impl Stream<Item = io::Result<Async<UnixStream>>> + Send + '_ {
stream::unfold(self, |listener| async move {
let res = listener.accept().await.map(|(stream, _)| stream);
Some((res, listener))
})
}
}
#[cfg(unix)]
impl TryFrom<std::os::unix::net::UnixListener> for Async<std::os::unix::net::UnixListener> {
type Error = io::Error;
fn try_from(listener: std::os::unix::net::UnixListener) -> io::Result<Self> {
Async::new(listener)
}
}
#[cfg(unix)]
impl Async<UnixStream> {
/// Creates a UDS stream connected to the specified path.
pub async fn connect<P: AsRef<Path>>(path: P) -> io::Result<Async<UnixStream>> {
// Begin async connect.
let socket = connect(SockAddr::unix(path)?, Domain::UNIX, None)?;
let stream = Async::new(UnixStream::from(socket))?;
// The stream becomes writable when connected.
stream.writable().await?;
// On Linux, it appears the socket may become writable even when connecting fails, so we
// must do an extra check here and see if the peer address is retrievable.
stream.get_ref().peer_addr()?;
Ok(stream)
}
/// Creates an unnamed pair of connected UDS stream sockets.
pub fn pair() -> io::Result<(Async<UnixStream>, Async<UnixStream>)> {
let (stream1, stream2) = UnixStream::pair()?;
Ok((Async::new(stream1)?, Async::new(stream2)?))
}
}
#[cfg(unix)]
impl TryFrom<std::os::unix::net::UnixStream> for Async<std::os::unix::net::UnixStream> {
type Error = io::Error;
fn try_from(stream: std::os::unix::net::UnixStream) -> io::Result<Self> {
Async::new(stream)
}
}
#[cfg(unix)]
impl Async<UnixDatagram> {
/// Creates a UDS datagram socket bound to the specified path.
pub fn bind<P: AsRef<Path>>(path: P) -> io::Result<Async<UnixDatagram>> {
let path = path.as_ref().to_owned();
Async::new(UnixDatagram::bind(path)?)
}
/// Creates a UDS datagram socket not bound to any address.
pub fn unbound() -> io::Result<Async<UnixDatagram>> {
Async::new(UnixDatagram::unbound()?)
}
/// Creates an unnamed pair of connected Unix datagram sockets.
pub fn pair() -> io::Result<(Async<UnixDatagram>, Async<UnixDatagram>)> {
let (socket1, socket2) = UnixDatagram::pair()?;
Ok((Async::new(socket1)?, Async::new(socket2)?))
}
/// Receives data from the socket.
///
/// Returns the number of bytes read and the address the message came from.
pub async fn recv_from(&self, buf: &mut [u8]) -> io::Result<(usize, UnixSocketAddr)> {
self.read_with(|io| io.recv_from(buf)).await
}
/// Sends data to the specified address.
///
/// Returns the number of bytes written.
pub async fn send_to<P: AsRef<Path>>(&self, buf: &[u8], path: P) -> io::Result<usize> {
self.write_with(|io| io.send_to(buf, &path)).await
}
/// Receives data from the connected peer.
///
/// Returns the number of bytes read and the address the message came from.
///
/// The [`connect`][`UnixDatagram::connect()`] method connects this socket to a remote address.
/// This method will fail if the socket is not connected.
pub async fn recv(&self, buf: &mut [u8]) -> io::Result<usize> {
self.read_with(|io| io.recv(buf)).await
}
/// Sends data to the connected peer.
///
/// Returns the number of bytes written.
///
/// The [`connect`][`UnixDatagram::connect()`] method connects this socket to a remote address.
/// This method will fail if the socket is not connected.
pub async fn send(&self, buf: &[u8]) -> io::Result<usize> {
self.write_with(|io| io.send(buf)).await
}
}
#[cfg(unix)]
impl TryFrom<std::os::unix::net::UnixDatagram> for Async<std::os::unix::net::UnixDatagram> {
type Error = io::Error;
fn try_from(socket: std::os::unix::net::UnixDatagram) -> io::Result<Self> {
Async::new(socket)
}
}
/// Polls a future once, waits for a wakeup, and then optimistically assumes the future is ready.
async fn optimistic(fut: impl Future<Output = io::Result<()>>) -> io::Result<()> {
let mut polled = false;
pin_mut!(fut);
future::poll_fn(|cx| {
if !polled {
polled = true;
fut.as_mut().poll(cx)
} else {
Poll::Ready(Ok(()))
}
})
.await
}
fn connect(addr: SockAddr, domain: Domain, protocol: Option<Protocol>) -> io::Result<Socket> {
let sock_type = Type::STREAM;
#[cfg(any(
target_os = "android",
target_os = "dragonfly",
target_os = "freebsd",
target_os = "fuchsia",
target_os = "illumos",
target_os = "linux",
target_os = "netbsd",
target_os = "openbsd"
))]
// If we can, set nonblocking at socket creation for unix
let sock_type = sock_type.nonblocking();
// This automatically handles cloexec on unix, no_inherit on windows and nosigpipe on macos
let socket = Socket::new(domain, sock_type, protocol)?;
#[cfg(not(any(
target_os = "android",
target_os = "dragonfly",
target_os = "freebsd",
target_os = "fuchsia",
target_os = "illumos",
target_os = "linux",
target_os = "netbsd",
target_os = "openbsd"
)))]
// If the current platform doesn't support nonblocking at creation, enable it after creation
socket.set_nonblocking(true)?;
match socket.connect(&addr) {
Ok(_) => {}
#[cfg(unix)]
Err(err) if err.raw_os_error() == Some(libc::EINPROGRESS) => {}
Err(err) if err.kind() == io::ErrorKind::WouldBlock => {}
Err(err) => return Err(err),
}
Ok(socket)
}

View file

@ -1,30 +1,19 @@
// Copyright (C) 2018-2020 Sebastian Dröge <sebastian@centricular.com>
// Copyright (C) 2019-2021 François Laignel <fengalin@free.fr>
//
// This library is free software; you can redistribute it and/or
// modify it under the terms of the GNU Library General Public
// License as published by the Free Software Foundation; either
// version 2 of the License, or (at your option) any later version.
//
// This library is distributed in the hope that it will be useful,
// but WITHOUT ANY WARRANTY; without even the implied warranty of
// MERCHANTABILITY or FITNESS FOR A PARTICULAR PURPOSE. See the GNU
// Library General Public License for more details.
//
// You should have received a copy of the GNU Library General Public
// License along with this library; if not, write to the
// Free Software Foundation, Inc., 51 Franklin Street, Suite 500,
// Boston, MA 02110-1335, USA.
// Take a look at the license at the top of the repository in the LICENSE file.
use futures::prelude::*;
use gst::{gst_debug, gst_trace};
use gst::{gst_debug, gst_error, gst_trace, gst_warning};
use once_cell::sync::Lazy;
use std::collections::HashMap;
use std::io;
use std::pin::Pin;
use std::sync::{Arc, Mutex};
use std::task::{self, Poll};
use std::time::Duration;
use super::{Handle, HandleWeak, JoinHandle, Scheduler, SubTaskOutput, TaskId};
@ -55,7 +44,12 @@ static CONTEXTS: Lazy<Mutex<HashMap<Arc<str>, ContextWeak>>> =
///
/// Note that you must not pass any futures here that wait for the currently active task in one way
/// or another as this would deadlock!
pub fn block_on_or_add_sub_task<Fut: Future + Send + 'static>(future: Fut) -> Option<Fut::Output> {
#[track_caller]
pub fn block_on_or_add_sub_task<Fut>(future: Fut) -> Option<Fut::Output>
where
Fut: Future + Send + 'static,
Fut::Output: Send + 'static,
{
if let Some((cur_context, cur_task_id)) = Context::current_task() {
gst_debug!(
RUNTIME_CAT,
@ -84,18 +78,45 @@ pub fn block_on_or_add_sub_task<Fut: Future + Send + 'static>(future: Fut) -> Op
/// # Panics
///
/// This function panics if called within a [`Context`] thread.
pub fn block_on<F: Future>(future: F) -> F::Output {
assert!(!Context::is_context_thread());
#[track_caller]
pub fn block_on<Fut>(future: Fut) -> Fut::Output
where
Fut: Future + Send + 'static,
Fut::Output: Send + 'static,
{
if let Some(context) = Context::current() {
let msg = format!("Attempt to block within Context {}", context.name());
gst_error!(RUNTIME_CAT, "{}", msg);
panic!("{}", msg);
}
// Not running in a Context thread so we can block
gst_debug!(RUNTIME_CAT, "Blocking on new dummy context");
Scheduler::block_on(future)
}
/// Yields execution back to the runtime
/// Yields execution back to the runtime.
#[inline]
pub async fn yield_now() {
tokio::task::yield_now().await;
pub fn yield_now() -> YieldNow {
YieldNow::default()
}
#[derive(Debug, Default)]
#[must_use = "futures do nothing unless you `.await` or poll them"]
pub struct YieldNow(bool);
impl Future for YieldNow {
type Output = ();
fn poll(mut self: Pin<&mut Self>, cx: &mut task::Context<'_>) -> Poll<Self::Output> {
if !self.0 {
self.0 = true;
cx.waker().wake_by_ref();
Poll::Pending
} else {
Poll::Ready(())
}
}
}
#[derive(Clone, Debug)]
@ -178,10 +199,39 @@ impl Context {
Scheduler::current().map(Context).zip(TaskId::current())
}
pub fn enter<F, R>(&self, f: F) -> R
/// Executes the provided function relatively to this [`Context`].
///
/// Usefull to initialze i/o sources and timers from outside
/// of a [`Context`].
///
/// # Panic
///
/// This will block current thread and would panic if run
/// from the [`Context`].
#[track_caller]
pub fn enter<F, O>(&self, f: F) -> O
where
F: FnOnce() -> R,
F: FnOnce() -> O + Send + 'static,
O: Send + 'static,
{
if let Some(cur) = Context::current().as_ref() {
if cur == self {
panic!(
"Attempt to enter Context {} within itself, this would deadlock",
self.name()
);
} else {
gst_warning!(
RUNTIME_CAT,
"Entering Context {} within {}",
self.name(),
cur.name()
);
}
} else {
gst_debug!(RUNTIME_CAT, "Entering Context {}", self.name());
}
self.0.enter(f)
}
@ -190,15 +240,15 @@ impl Context {
Fut: Future + Send + 'static,
Fut::Output: Send + 'static,
{
self.0.spawn(future, false)
self.0.spawn(future)
}
pub fn awake_and_spawn<Fut>(&self, future: Fut) -> JoinHandle<Fut::Output>
pub fn spawn_and_awake<Fut>(&self, future: Fut) -> JoinHandle<Fut::Output>
where
Fut: Future + Send + 'static,
Fut::Output: Send + 'static,
{
self.0.spawn(future, true)
self.0.spawn_and_awake(future)
}
pub fn current_has_sub_tasks() -> bool {
@ -256,6 +306,7 @@ mod tests {
use super::super::Scheduler;
use super::Context;
use crate::runtime::Async;
type Item = i32;
@ -301,24 +352,27 @@ mod tests {
#[test]
fn context_task_id() {
use super::TaskId;
gst::init().unwrap();
let context = Context::acquire("context_task_id", SLEEP_DURATION).unwrap();
let join_handle = context.spawn(async {
let (ctx, task_id) = Context::current_task().unwrap();
assert_eq!(ctx.name(), "context_task_id");
assert_eq!(task_id, super::TaskId(0));
assert_eq!(task_id, TaskId(0));
});
futures::executor::block_on(join_handle).unwrap();
// TaskId(0) is vacant again
let ctx_weak = context.downgrade();
let join_handle = context.spawn(async move {
let (_ctx, task_id) = Context::current_task().unwrap();
assert_eq!(task_id, super::TaskId(1));
assert_eq!(task_id, TaskId(0));
let res = Context::add_sub_task(async move {
let (_ctx, task_id) = Context::current_task().unwrap();
assert_eq!(task_id, super::TaskId(1));
assert_eq!(task_id, TaskId(0));
Ok(())
});
assert!(res.is_ok());
@ -328,18 +382,18 @@ mod tests {
.unwrap()
.spawn(async {
let (_ctx, task_id) = Context::current_task().unwrap();
assert_eq!(task_id, super::TaskId(2));
assert_eq!(task_id, TaskId(1));
let res = Context::add_sub_task(async move {
let (_ctx, task_id) = Context::current_task().unwrap();
assert_eq!(task_id, super::TaskId(2));
assert_eq!(task_id, TaskId(1));
Ok(())
});
assert!(res.is_ok());
assert!(Context::drain_sub_tasks().await.is_ok());
let (_ctx, task_id) = Context::current_task().unwrap();
assert_eq!(task_id, super::TaskId(2));
assert_eq!(task_id, TaskId(1));
})
.await
.unwrap();
@ -347,7 +401,7 @@ mod tests {
assert!(Context::drain_sub_tasks().await.is_ok());
let (_ctx, task_id) = Context::current_task().unwrap();
assert_eq!(task_id, super::TaskId(1));
assert_eq!(task_id, TaskId(0));
});
futures::executor::block_on(join_handle).unwrap();
}
@ -417,18 +471,17 @@ mod tests {
let bytes_sent = crate::runtime::executor::block_on(context.spawn(async {
let saddr = SocketAddr::new(IpAddr::V4(Ipv4Addr::LOCALHOST), 5001);
let socket = UdpSocket::bind(saddr).unwrap();
let mut socket = tokio::net::UdpSocket::from_std(socket).unwrap();
let saddr = SocketAddr::new(IpAddr::V4(Ipv4Addr::LOCALHOST), 4000);
let socket = Async::<UdpSocket>::bind(saddr).unwrap();
let saddr = SocketAddr::new(IpAddr::V4(Ipv4Addr::LOCALHOST), 4001);
socket.send_to(&[0; 10], saddr).await.unwrap()
}))
.unwrap();
assert_eq!(bytes_sent, 10);
let elapsed = crate::runtime::executor::block_on(context.spawn(async {
let now = Instant::now();
let start = Instant::now();
crate::runtime::time::delay_for(DELAY).await;
now.elapsed()
start.elapsed()
}))
.unwrap();
// Due to throttling, `Delay` may be fired earlier
@ -436,16 +489,19 @@ mod tests {
}
#[test]
#[should_panic]
fn block_on_from_context() {
gst::init().unwrap();
let context = Context::acquire("block_on_from_context", SLEEP_DURATION).unwrap();
let join_handle = context.spawn(async {
crate::runtime::executor::block_on(async {
crate::runtime::time::delay_for(DELAY).await;
});
});
// Panic: attempt to `runtime::executor::block_on` within a `Context` thread
let join_handle = context.spawn(async {
crate::runtime::executor::block_on(crate::runtime::time::delay_for(DELAY));
});
// Panic: task has failed
// (enforced by `async-task`, see comment in `Future` impl for `JoinHanlde`).
futures::executor::block_on(join_handle).unwrap_err();
}
@ -454,26 +510,22 @@ mod tests {
gst::init().unwrap();
let elapsed = crate::runtime::executor::block_on(async {
let context = Context::acquire("enter_context_from_tokio", SLEEP_DURATION).unwrap();
let mut socket = context
let context = Context::acquire("enter_context_from_executor", SLEEP_DURATION).unwrap();
let socket = context
.enter(|| {
let saddr = SocketAddr::new(IpAddr::V4(Ipv4Addr::LOCALHOST), 5002);
let socket = UdpSocket::bind(saddr).unwrap();
tokio::net::UdpSocket::from_std(socket)
Async::<UdpSocket>::bind(saddr)
})
.unwrap();
let saddr = SocketAddr::new(IpAddr::V4(Ipv4Addr::LOCALHOST), 4000);
let saddr = SocketAddr::new(IpAddr::V4(Ipv4Addr::LOCALHOST), 4002);
let bytes_sent = socket.send_to(&[0; 10], saddr).await.unwrap();
assert_eq!(bytes_sent, 10);
context.enter(|| {
futures::executor::block_on(async {
let now = Instant::now();
crate::runtime::time::delay_for(DELAY).await;
now.elapsed()
})
})
let (start, timer) =
context.enter(|| (Instant::now(), crate::runtime::time::delay_for(DELAY)));
timer.await;
start.elapsed()
});
// Due to throttling, `Delay` may be fired earlier
@ -485,24 +537,22 @@ mod tests {
gst::init().unwrap();
let context = Context::acquire("enter_context_from_sync", SLEEP_DURATION).unwrap();
let mut socket = context
let socket = context
.enter(|| {
let saddr = SocketAddr::new(IpAddr::V4(Ipv4Addr::LOCALHOST), 5003);
let socket = UdpSocket::bind(saddr).unwrap();
tokio::net::UdpSocket::from_std(socket)
Async::<UdpSocket>::bind(saddr)
})
.unwrap();
let saddr = SocketAddr::new(IpAddr::V4(Ipv4Addr::LOCALHOST), 4000);
let saddr = SocketAddr::new(IpAddr::V4(Ipv4Addr::LOCALHOST), 4003);
let bytes_sent = futures::executor::block_on(socket.send_to(&[0; 10], saddr)).unwrap();
assert_eq!(bytes_sent, 10);
let elapsed = context.enter(|| {
futures::executor::block_on(async {
let now = Instant::now();
crate::runtime::time::delay_for(DELAY).await;
now.elapsed()
})
let (start, timer) =
context.enter(|| (Instant::now(), crate::runtime::time::delay_for(DELAY)));
let elapsed = crate::runtime::executor::block_on(async move {
timer.await;
start.elapsed()
});
// Due to throttling, `Delay` may be fired earlier
assert!(elapsed + SLEEP_DURATION / 2 >= DELAY);

View file

@ -1,22 +1,8 @@
// Copyright (C) 2018-2020 Sebastian Dröge <sebastian@centricular.com>
// Copyright (C) 2019-2021 François Laignel <fengalin@free.fr>
//
// This library is free software; you can redistribute it and/or
// modify it under the terms of the GNU Library General Public
// License as published by the Free Software Foundation; either
// version 2 of the License, or (at your option) any later version.
//
// This library is distributed in the hope that it will be useful,
// but WITHOUT ANY WARRANTY; without even the implied warranty of
// MERCHANTABILITY or FITNESS FOR A PARTICULAR PURPOSE. See the GNU
// Library General Public License for more details.
//
// You should have received a copy of the GNU Library General Public
// License along with this library; if not, write to the
// Free Software Foundation, Inc., 51 Franklin Street, Suite 500,
// Boston, MA 02110-1335, USA.
// Take a look at the license at the top of the repository in the LICENSE file.
use futures::channel::oneshot;
use futures::prelude::*;
use std::fmt;
@ -26,54 +12,56 @@ use std::task::Poll;
use super::context::Context;
use super::TaskId;
use super::{Handle, HandleWeak, Scheduler};
use super::{Handle, Scheduler};
#[derive(Debug)]
pub struct JoinError(TaskId);
impl fmt::Display for JoinError {
fn fmt(&self, fmt: &mut fmt::Formatter<'_>) -> fmt::Result {
write!(fmt, "{:?} was Canceled", self.0)
write!(fmt, "{:?} was cancelled", self.0)
}
}
impl std::error::Error for JoinError {}
pub struct JoinHandle<T> {
receiver: oneshot::Receiver<T>,
handle: HandleWeak,
task: Option<async_task::Task<T>>,
task_id: TaskId,
scheduler: Handle,
}
unsafe impl<T: Send> Send for JoinHandle<T> {}
unsafe impl<T: Send> Sync for JoinHandle<T> {}
impl<T> JoinHandle<T> {
pub(super) fn new(receiver: oneshot::Receiver<T>, handle: &Handle, task_id: TaskId) -> Self {
pub(super) fn new(task_id: TaskId, task: async_task::Task<T>, scheduler: &Handle) -> Self {
JoinHandle {
receiver,
handle: handle.downgrade(),
task: Some(task),
task_id,
scheduler: scheduler.clone(),
}
}
pub fn is_current(&self) -> bool {
if let Some((cur_scheduler, task_id)) = Scheduler::current().zip(TaskId::current()) {
self.handle.upgrade().map_or(false, |self_scheduler| {
self_scheduler == cur_scheduler && task_id == self.task_id
})
cur_scheduler == self.scheduler && task_id == self.task_id
} else {
false
}
}
pub fn context(&self) -> Option<Context> {
self.handle.upgrade().map(Context::from)
pub fn context(&self) -> Context {
Context::from(self.scheduler.clone())
}
pub fn task_id(&self) -> TaskId {
self.task_id
}
pub fn cancel(mut self) {
let _ = self.task.take().map(|task| task.cancel());
}
}
impl<T> Future for JoinHandle<T> {
@ -84,22 +72,30 @@ impl<T> Future for JoinHandle<T> {
panic!("Trying to join task {:?} from itself", self.as_ref());
}
self.as_mut()
.receiver
.poll_unpin(cx)
.map_err(|_| JoinError(self.task_id))
if let Some(task) = self.as_mut().task.as_mut() {
// Unfortunately, we can't detect whether the task has panicked
// because the `async_task::Task` `Future` implementation
// `expect`s and we can't `panic::catch_unwind` here because of `&mut cx`.
// One solution for this would be to use our own `async_task` impl.
task.poll_unpin(cx).map(Ok)
} else {
Poll::Ready(Err(JoinError(self.task_id)))
}
}
}
impl<T> Drop for JoinHandle<T> {
fn drop(&mut self) {
if let Some(task) = self.task.take() {
task.detach();
}
}
}
impl<T> fmt::Debug for JoinHandle<T> {
fn fmt(&self, fmt: &mut fmt::Formatter<'_>) -> fmt::Result {
let context_name = self
.handle
.upgrade()
.map(|handle| handle.context_name().to_owned());
fmt.debug_struct("JoinHandle")
.field("context", &context_name)
.field("context", &self.scheduler.context_name())
.field("task_id", &self.task_id)
.finish()
}

View file

@ -1,20 +1,7 @@
// Copyright (C) 2018-2020 Sebastian Dröge <sebastian@centricular.com>
// Copyright (C) 2019-2021 François Laignel <fengalin@free.fr>
//
// This library is free software; you can redistribute it and/or
// modify it under the terms of the GNU Library General Public
// License as published by the Free Software Foundation; either
// version 2 of the License, or (at your option) any later version.
//
// This library is distributed in the hope that it will be useful,
// but WITHOUT ANY WARRANTY; without even the implied warranty of
// MERCHANTABILITY or FITNESS FOR A PARTICULAR PURPOSE. See the GNU
// Library General Public License for more details.
//
// You should have received a copy of the GNU Library General Public
// License along with this library; if not, write to the
// Free Software Foundation, Inc., 51 Franklin Street, Suite 500,
// Boston, MA 02110-1335, USA.
// Take a look at the license at the top of the repository in the LICENSE file.
//! The `Executor` for the `threadshare` GStreamer plugins framework.
//!
@ -33,18 +20,29 @@
//! [`PadSrc`]: ../pad/struct.PadSrc.html
//! [`PadSink`]: ../pad/struct.PadSink.html
pub mod async_wrapper;
pub use async_wrapper::Async;
mod context;
pub use context::{block_on, block_on_or_add_sub_task, yield_now, Context};
mod scheduler;
use scheduler::{Handle, HandleWeak, Scheduler};
mod join;
pub use join::JoinHandle;
pub mod reactor;
use reactor::{Reactor, Readable, ReadableOwned, Source, Writable, WritableOwned};
// We need the `Mutex<bool>` to work in pair with `Condvar`.
#[allow(clippy::mutex_atomic)]
mod scheduler;
use scheduler::{Handle, HandleWeak, Scheduler};
mod task;
pub use task::{SubTaskOutput, TaskId};
pub mod timer;
pub use timer::Timer;
struct CallOnDrop<F: FnOnce()>(Option<F>);
impl<F: FnOnce()> CallOnDrop<F> {

View file

@ -0,0 +1,675 @@
// This is based on https://github.com/smol-rs/async-io
// with adaptations by:
//
// Copyright (C) 2021 François Laignel <fengalin@free.fr>
//
// Take a look at the license at the top of the repository in the LICENSE file.
use concurrent_queue::ConcurrentQueue;
use futures::ready;
use gst::{gst_trace, gst_warning};
use polling::{Event, Poller};
use slab::Slab;
use std::borrow::Borrow;
use std::cell::RefCell;
use std::collections::BTreeMap;
use std::fmt;
use std::future::Future;
use std::io;
use std::marker::PhantomData;
use std::mem;
#[cfg(unix)]
use std::os::unix::io::RawFd;
#[cfg(windows)]
use std::os::windows::io::RawSocket;
use std::panic;
use std::pin::Pin;
use std::sync::atomic::{AtomicUsize, Ordering};
use std::sync::{Arc, Mutex};
use std::task::{Context, Poll, Waker};
use std::time::{Duration, Instant};
use crate::runtime::{Async, RUNTIME_CAT};
const READ: usize = 0;
const WRITE: usize = 1;
thread_local! {
static CURRENT_REACTOR: RefCell<Option<Reactor>> = RefCell::new(None);
}
#[derive(Debug)]
pub(super) struct Reactor {
/// Portable bindings to epoll/kqueue/event ports/wepoll.
///
/// This is where I/O is polled, producing I/O events.
poller: Poller,
/// Ticker bumped before polling.
///
/// This is useful for checking what is the current "round" of `ReactorLock::react()` when
/// synchronizing things in `Source::readable()` and `Source::writable()`. Both of those
/// methods must make sure they don't receive stale I/O events - they only accept events from a
/// fresh "round" of `ReactorLock::react()`.
ticker: AtomicUsize,
/// Half max throttling duration, needed to fire timers.
half_max_throttling: Duration,
/// Registered sources.
sources: Slab<Arc<Source>>,
/// Temporary storage for I/O events when polling the reactor.
///
/// Holding a lock on this event list implies the exclusive right to poll I/O.
events: Vec<Event>,
/// An ordered map of registered timers.
///
/// Timers are in the order in which they fire. The `usize` in this type is a timer ID used to
/// distinguish timers that fire at the same time. The `Waker` represents the task awaiting the
/// timer.
timers: BTreeMap<(Instant, usize), Waker>,
/// A queue of timer operations (insert and remove).
///
/// When inserting or removing a timer, we don't process it immediately - we just push it into
/// this queue. Timers actually get processed when the queue fills up or the reactor is polled.
timer_ops: ConcurrentQueue<TimerOp>,
}
impl Reactor {
fn new(max_throttling: Duration) -> Self {
Reactor {
poller: Poller::new().expect("cannot initialize I/O event notification"),
ticker: AtomicUsize::new(0),
half_max_throttling: max_throttling / 2 + Duration::from_nanos(1),
sources: Slab::new(),
events: Vec::new(),
timers: BTreeMap::new(),
timer_ops: ConcurrentQueue::bounded(1000),
}
}
/// Initializes the reactor for current thread.
pub fn init(max_throttling: Duration) {
CURRENT_REACTOR.with(|cur| {
let mut cur = cur.borrow_mut();
if cur.is_none() {
*cur = Some(Reactor::new(max_throttling));
}
})
}
pub fn close() {
let _ = CURRENT_REACTOR.try_with(|cur_reactor| {
*cur_reactor.borrow_mut() = None;
});
}
/// Executes the function with current thread's reactor as ref.
///
/// # Panics
///
/// Panics if:
///
/// - The Reactor is not initialized, i.e. if
/// current thread is not a [`Context`] thread.
/// - The Reactor is already mutably borrowed.
///
/// Use [`Context::enter`] to register i/o sources
/// or timers from a different thread.
#[track_caller]
pub fn with<F, R>(f: F) -> R
where
F: FnOnce(&Reactor) -> R,
{
CURRENT_REACTOR.with(|reactor| {
f(reactor
.borrow()
.as_ref()
.expect("Not running in a Context."))
})
}
/// Executes the function with current thread's reactor as mutable.
///
/// # Panics
///
/// Panics if:
///
/// - The Reactor is not initialized, i.e. if
/// current thread is not a [`Context`] thread.
/// - The Reactor is already mutably borrowed.
///
/// Use [`Context::enter`] to register i/o sources
/// or timers from a different thread.
#[track_caller]
pub fn with_mut<F, R>(f: F) -> R
where
F: FnOnce(&mut Reactor) -> R,
{
CURRENT_REACTOR.with(|reactor| {
f(reactor
.borrow_mut()
.as_mut()
.expect("Not running in a Context."))
})
}
/// Returns the current ticker.
pub fn ticker(&self) -> usize {
self.ticker.load(Ordering::SeqCst)
}
pub fn half_max_throttling(&self) -> Duration {
self.half_max_throttling
}
/// Registers an I/O source in the reactor.
pub fn insert_io(
&mut self,
#[cfg(unix)] raw: RawFd,
#[cfg(windows)] raw: RawSocket,
) -> io::Result<Arc<Source>> {
// Create an I/O source for this file descriptor.
let source = {
let key = self.sources.vacant_entry().key();
let source = Arc::new(Source {
raw,
key,
state: Default::default(),
});
self.sources.insert(source.clone());
source
};
// Register the file descriptor.
if let Err(err) = self.poller.add(raw, Event::none(source.key)) {
self.sources.remove(source.key);
return Err(err);
}
Ok(source)
}
/// Deregisters an I/O source from the reactor.
pub fn remove_io(&mut self, source: &Source) -> io::Result<()> {
self.sources.remove(source.key);
self.poller.delete(source.raw)
}
/// Registers a timer in the reactor.
///
/// Returns the inserted timer's ID.
pub fn insert_timer(&mut self, when: Instant, waker: &Waker) -> usize {
// Generate a new timer ID.
static ID_GENERATOR: AtomicUsize = AtomicUsize::new(1);
let id = ID_GENERATOR.fetch_add(1, Ordering::Relaxed);
// Push an insert operation.
while self
.timer_ops
.push(TimerOp::Insert(when, id, waker.clone()))
.is_err()
{
// If the queue is full, drain it and try again.
gst_warning!(RUNTIME_CAT, "react: timer_ops is full");
self.process_timer_ops();
}
id
}
/// Deregisters a timer from the reactor.
pub fn remove_timer(&mut self, when: Instant, id: usize) {
// Push a remove operation.
while self.timer_ops.push(TimerOp::Remove(when, id)).is_err() {
gst_warning!(RUNTIME_CAT, "react: timer_ops is full");
// If the queue is full, drain it and try again.
self.process_timer_ops();
}
}
/// Processes ready timers and extends the list of wakers to wake.
///
/// Returns the duration until the next timer before this method was called.
fn process_timers(&mut self, wakers: &mut Vec<Waker>) {
self.process_timer_ops();
let now = Instant::now();
// Split timers into ready and pending timers.
//
// Careful to split just *after* `now`, so that a timer set for exactly `now` is considered
// ready.
let pending = self.timers.split_off(&(now + self.half_max_throttling, 0));
let ready = mem::replace(&mut self.timers, pending);
// Add wakers to the list.
if !ready.is_empty() {
gst_trace!(RUNTIME_CAT, "process_timers: {} ready wakers", ready.len());
for (_, waker) in ready {
wakers.push(waker);
}
}
}
/// Processes queued timer operations.
fn process_timer_ops(&mut self) {
// Process only as much as fits into the queue, or else this loop could in theory run
// forever.
for _ in 0..self.timer_ops.capacity().unwrap() {
match self.timer_ops.pop() {
Ok(TimerOp::Insert(when, id, waker)) => {
self.timers.insert((when, id), waker);
}
Ok(TimerOp::Remove(when, id)) => {
self.timers.remove(&(when, id));
}
Err(_) => break,
}
}
}
/// Processes new events.
pub fn react(&mut self) -> io::Result<()> {
let mut wakers = Vec::new();
// Process ready timers.
self.process_timers(&mut wakers);
// Bump the ticker before polling I/O.
let tick = self.ticker.fetch_add(1, Ordering::SeqCst).wrapping_add(1);
self.events.clear();
// Block on I/O events.
let res = match self.poller.wait(&mut self.events, Some(Duration::ZERO)) {
// No I/O events occurred.
Ok(0) => Ok(()),
// At least one I/O event occurred.
Ok(_) => {
for ev in self.events.iter() {
// Check if there is a source in the table with this key.
if let Some(source) = self.sources.get(ev.key) {
let mut state = source.state.lock().unwrap();
// Collect wakers if a writability event was emitted.
for &(dir, emitted) in &[(WRITE, ev.writable), (READ, ev.readable)] {
if emitted {
state[dir].tick = tick;
state[dir].drain_into(&mut wakers);
}
}
// Re-register if there are still writers or readers. The can happen if
// e.g. we were previously interested in both readability and writability,
// but only one of them was emitted.
if !state[READ].is_empty() || !state[WRITE].is_empty() {
self.poller.modify(
source.raw,
Event {
key: source.key,
readable: !state[READ].is_empty(),
writable: !state[WRITE].is_empty(),
},
)?;
}
}
}
Ok(())
}
// The syscall was interrupted.
Err(err) if err.kind() == io::ErrorKind::Interrupted => Ok(()),
// An actual error occureed.
Err(err) => Err(err),
};
// Wake up ready tasks.
if !wakers.is_empty() {
gst_trace!(RUNTIME_CAT, "react: {} ready wakers", wakers.len());
for waker in wakers {
// Don't let a panicking waker blow everything up.
panic::catch_unwind(|| waker.wake()).ok();
}
}
res
}
}
/// A single timer operation.
enum TimerOp {
Insert(Instant, usize, Waker),
Remove(Instant, usize),
}
/// A registered source of I/O events.
#[derive(Debug)]
pub(super) struct Source {
/// Raw file descriptor on Unix platforms.
#[cfg(unix)]
pub(super) raw: RawFd,
/// Raw socket handle on Windows.
#[cfg(windows)]
pub(super) raw: RawSocket,
/// The key of this source obtained during registration.
key: usize,
/// Inner state with registered wakers.
state: Mutex<[Direction; 2]>,
}
/// A read or write direction.
#[derive(Debug, Default)]
struct Direction {
/// Last reactor tick that delivered an event.
tick: usize,
/// Ticks remembered by `Async::poll_readable()` or `Async::poll_writable()`.
ticks: Option<(usize, usize)>,
/// Waker stored by `Async::poll_readable()` or `Async::poll_writable()`.
waker: Option<Waker>,
/// Wakers of tasks waiting for the next event.
///
/// Registered by `Async::readable()` and `Async::writable()`.
wakers: Slab<Option<Waker>>,
}
impl Direction {
/// Returns `true` if there are no wakers interested in this direction.
fn is_empty(&self) -> bool {
self.waker.is_none() && self.wakers.iter().all(|(_, opt)| opt.is_none())
}
/// Moves all wakers into a `Vec`.
fn drain_into(&mut self, dst: &mut Vec<Waker>) {
if let Some(w) = self.waker.take() {
dst.push(w);
}
for (_, opt) in self.wakers.iter_mut() {
if let Some(w) = opt.take() {
dst.push(w);
}
}
}
}
impl Source {
/// Polls the I/O source for readability.
pub fn poll_readable(&self, cx: &mut Context<'_>) -> Poll<io::Result<()>> {
self.poll_ready(READ, cx)
}
/// Polls the I/O source for writability.
pub fn poll_writable(&self, cx: &mut Context<'_>) -> Poll<io::Result<()>> {
self.poll_ready(WRITE, cx)
}
/// Registers a waker from `poll_readable()` or `poll_writable()`.
///
/// If a different waker is already registered, it gets replaced and woken.
fn poll_ready(&self, dir: usize, cx: &mut Context<'_>) -> Poll<io::Result<()>> {
let mut state = self.state.lock().unwrap();
// Check if the reactor has delivered an event.
if let Some((a, b)) = state[dir].ticks {
// If `state[dir].tick` has changed to a value other than the old reactor tick,
// that means a newer reactor tick has delivered an event.
if state[dir].tick != a && state[dir].tick != b {
state[dir].ticks = None;
return Poll::Ready(Ok(()));
}
}
let was_empty = state[dir].is_empty();
// Register the current task's waker.
if let Some(w) = state[dir].waker.take() {
if w.will_wake(cx.waker()) {
state[dir].waker = Some(w);
return Poll::Pending;
}
// Wake the previous waker because it's going to get replaced.
panic::catch_unwind(|| w.wake()).ok();
}
Reactor::with(|reactor| {
state[dir].waker = Some(cx.waker().clone());
state[dir].ticks = Some((reactor.ticker(), state[dir].tick));
// Update interest in this I/O handle.
if was_empty {
reactor.poller.modify(
self.raw,
Event {
key: self.key,
readable: !state[READ].is_empty(),
writable: !state[WRITE].is_empty(),
},
)?;
}
Poll::Pending
})
}
/// Waits until the I/O source is readable.
pub fn readable<T>(handle: &Async<T>) -> Readable<'_, T> {
Readable(Self::ready(handle, READ))
}
/// Waits until the I/O source is readable.
pub fn readable_owned<T>(handle: Arc<Async<T>>) -> ReadableOwned<T> {
ReadableOwned(Self::ready(handle, READ))
}
/// Waits until the I/O source is writable.
pub fn writable<T>(handle: &Async<T>) -> Writable<'_, T> {
Writable(Self::ready(handle, WRITE))
}
/// Waits until the I/O source is writable.
pub fn writable_owned<T>(handle: Arc<Async<T>>) -> WritableOwned<T> {
WritableOwned(Self::ready(handle, WRITE))
}
/// Waits until the I/O source is readable or writable.
fn ready<H: Borrow<Async<T>> + Clone, T>(handle: H, dir: usize) -> Ready<H, T> {
Ready {
handle,
dir,
ticks: None,
index: None,
_guard: None,
}
}
}
/// Future for [`Async::readable`](crate::runtime::Async::readable).
#[must_use = "futures do nothing unless you `.await` or poll them"]
pub struct Readable<'a, T>(Ready<&'a Async<T>, T>);
impl<T> Future for Readable<'_, T> {
type Output = io::Result<()>;
fn poll(mut self: Pin<&mut Self>, cx: &mut Context<'_>) -> Poll<Self::Output> {
ready!(Pin::new(&mut self.0).poll(cx))?;
gst_trace!(RUNTIME_CAT, "readable: fd={}", self.0.handle.source.raw);
Poll::Ready(Ok(()))
}
}
impl<T> fmt::Debug for Readable<'_, T> {
fn fmt(&self, f: &mut fmt::Formatter<'_>) -> fmt::Result {
f.debug_struct("Readable").finish()
}
}
/// Future for [`Async::readable_owned`](crate::runtime::Async::readable_owned).
#[must_use = "futures do nothing unless you `.await` or poll them"]
pub struct ReadableOwned<T>(Ready<Arc<Async<T>>, T>);
impl<T> Future for ReadableOwned<T> {
type Output = io::Result<()>;
fn poll(mut self: Pin<&mut Self>, cx: &mut Context<'_>) -> Poll<Self::Output> {
ready!(Pin::new(&mut self.0).poll(cx))?;
gst_trace!(
RUNTIME_CAT,
"readable_owned: fd={}",
self.0.handle.source.raw
);
Poll::Ready(Ok(()))
}
}
impl<T> fmt::Debug for ReadableOwned<T> {
fn fmt(&self, f: &mut fmt::Formatter<'_>) -> fmt::Result {
f.debug_struct("ReadableOwned").finish()
}
}
/// Future for [`Async::writable`](crate::runtime::Async::writable).
#[must_use = "futures do nothing unless you `.await` or poll them"]
pub struct Writable<'a, T>(Ready<&'a Async<T>, T>);
impl<T> Future for Writable<'_, T> {
type Output = io::Result<()>;
fn poll(mut self: Pin<&mut Self>, cx: &mut Context<'_>) -> Poll<Self::Output> {
ready!(Pin::new(&mut self.0).poll(cx))?;
gst_trace!(RUNTIME_CAT, "writable: fd={}", self.0.handle.source.raw);
Poll::Ready(Ok(()))
}
}
impl<T> fmt::Debug for Writable<'_, T> {
fn fmt(&self, f: &mut fmt::Formatter<'_>) -> fmt::Result {
f.debug_struct("Writable").finish()
}
}
/// Future for [`Async::writable_owned`](crate::runtime::Async::writable_owned).
#[must_use = "futures do nothing unless you `.await` or poll them"]
pub struct WritableOwned<T>(Ready<Arc<Async<T>>, T>);
impl<T> Future for WritableOwned<T> {
type Output = io::Result<()>;
fn poll(mut self: Pin<&mut Self>, cx: &mut Context<'_>) -> Poll<Self::Output> {
ready!(Pin::new(&mut self.0).poll(cx))?;
gst_trace!(
RUNTIME_CAT,
"writable_owned: fd={}",
self.0.handle.source.raw
);
Poll::Ready(Ok(()))
}
}
impl<T> fmt::Debug for WritableOwned<T> {
fn fmt(&self, f: &mut fmt::Formatter<'_>) -> fmt::Result {
f.debug_struct("WritableOwned").finish()
}
}
struct Ready<H: Borrow<Async<T>>, T> {
handle: H,
dir: usize,
ticks: Option<(usize, usize)>,
index: Option<usize>,
_guard: Option<RemoveOnDrop<H, T>>,
}
impl<H: Borrow<Async<T>>, T> Unpin for Ready<H, T> {}
impl<H: Borrow<Async<T>> + Clone, T> Future for Ready<H, T> {
type Output = io::Result<()>;
fn poll(mut self: Pin<&mut Self>, cx: &mut Context<'_>) -> Poll<Self::Output> {
let Self {
ref handle,
dir,
ticks,
index,
_guard,
..
} = &mut *self;
let mut state = handle.borrow().source.state.lock().unwrap();
// Check if the reactor has delivered an event.
if let Some((a, b)) = *ticks {
// If `state[dir].tick` has changed to a value other than the old reactor tick,
// that means a newer reactor tick has delivered an event.
if state[*dir].tick != a && state[*dir].tick != b {
return Poll::Ready(Ok(()));
}
}
let was_empty = state[*dir].is_empty();
Reactor::with(|reactor| {
// Register the current task's waker.
let i = match *index {
Some(i) => i,
None => {
let i = state[*dir].wakers.insert(None);
*_guard = Some(RemoveOnDrop {
handle: handle.clone(),
dir: *dir,
key: i,
_marker: PhantomData,
});
*index = Some(i);
*ticks = Some((reactor.ticker(), state[*dir].tick));
i
}
};
state[*dir].wakers[i] = Some(cx.waker().clone());
// Update interest in this I/O handle.
if was_empty {
reactor.poller.modify(
handle.borrow().source.raw,
Event {
key: handle.borrow().source.key,
readable: !state[READ].is_empty(),
writable: !state[WRITE].is_empty(),
},
)?;
}
Poll::Pending
})
}
}
/// Remove waker when dropped.
struct RemoveOnDrop<H: Borrow<Async<T>>, T> {
handle: H,
dir: usize,
key: usize,
_marker: PhantomData<fn() -> T>,
}
impl<H: Borrow<Async<T>>, T> Drop for RemoveOnDrop<H, T> {
fn drop(&mut self) {
let mut state = self.handle.borrow().source.state.lock().unwrap();
let wakers = &mut state[self.dir].wakers;
if wakers.contains(self.key) {
wakers.remove(self.key);
}
}
}

View file

@ -1,119 +1,147 @@
// Copyright (C) 2018-2020 Sebastian Dröge <sebastian@centricular.com>
// Copyright (C) 2019-2021 François Laignel <fengalin@free.fr>
//
// This library is free software; you can redistribute it and/or
// modify it under the terms of the GNU Library General Public
// License as published by the Free Software Foundation; either
// version 2 of the License, or (at your option) any later version.
//
// This library is distributed in the hope that it will be useful,
// but WITHOUT ANY WARRANTY; without even the implied warranty of
// MERCHANTABILITY or FITNESS FOR A PARTICULAR PURPOSE. See the GNU
// Library General Public License for more details.
//
// You should have received a copy of the GNU Library General Public
// License along with this library; if not, write to the
// Free Software Foundation, Inc., 51 Franklin Street, Suite 500,
// Boston, MA 02110-1335, USA.
// Take a look at the license at the top of the repository in the LICENSE file.
use concurrent_queue::ConcurrentQueue;
use futures::channel::oneshot;
use futures::pin_mut;
use gio::glib::clone::Downgrade;
use gst::{gst_debug, gst_trace};
use gst::{gst_debug, gst_error, gst_trace, gst_warning};
use std::cell::RefCell;
use std::future::Future;
use std::panic;
use std::sync::mpsc as sync_mpsc;
use std::sync::{Arc, Mutex, Weak};
use std::sync::{Arc, Condvar, Mutex, Weak};
use std::task::Poll;
use std::thread;
use std::time::Duration;
use std::time::{Duration, Instant};
use super::task::{SubTaskOutput, TaskFuture, TaskId, TaskQueue};
use super::{CallOnDrop, JoinHandle};
use waker_fn::waker_fn;
use super::task::{SubTaskOutput, TaskId, TaskQueue};
use super::{CallOnDrop, JoinHandle, Reactor, Source};
use crate::runtime::RUNTIME_CAT;
thread_local! {
static CURRENT_SCHEDULER: RefCell<Option<Weak<Scheduler>>> = RefCell::new(None);
static CURRENT_SCHEDULER: RefCell<Option<HandleWeak>> = RefCell::new(None);
}
#[derive(Debug)]
struct CleanUpOps(Arc<Source>);
#[derive(Debug)]
pub(super) struct Scheduler {
context_name: Arc<str>,
max_throttling: Duration,
task_queue: Mutex<TaskQueue>,
rt_handle: Mutex<tokio::runtime::Handle>,
shutdown: Mutex<Option<SchedulerShutdown>>,
tasks: TaskQueue,
cleanup_ops: ConcurrentQueue<CleanUpOps>,
must_awake: Mutex<bool>,
must_awake_cvar: Condvar,
}
impl Scheduler {
pub const DUMMY_NAME: &'static str = "DUMMY";
pub fn start(context_name: &str, max_throttling: Duration) -> Handle {
let context_name = Arc::from(context_name);
// Name the thread so that it appears in panic messages.
let thread = thread::Builder::new().name(context_name.to_string());
let (handle_sender, handle_receiver) = sync_mpsc::channel();
let (shutdown_sender, shutdown_receiver) = oneshot::channel();
let context_name = Arc::from(context_name);
let thread_ctx_name = Arc::clone(&context_name);
let join = thread::spawn(move || {
gst_debug!(
RUNTIME_CAT,
"Started Scheduler thread for Context '{}'",
thread_ctx_name
);
let join = thread
.spawn(move || {
gst_debug!(
RUNTIME_CAT,
"Started Scheduler thread for Context {}",
thread_ctx_name
);
let (mut rt, handle) = Scheduler::init(thread_ctx_name, max_throttling);
handle_sender.send(handle.clone()).unwrap();
let handle = Scheduler::init(Arc::clone(&thread_ctx_name), max_throttling);
let this = Arc::clone(&handle.0.scheduler);
handle_sender.send(handle.clone()).unwrap();
let _ = rt.block_on(shutdown_receiver);
});
match this.block_on_priv(shutdown_receiver) {
Ok(_) => {
gst_debug!(
RUNTIME_CAT,
"Scheduler thread shut down for Context {}",
thread_ctx_name
);
}
Err(e) => {
gst_error!(
RUNTIME_CAT,
"Scheduler thread shut down due to an error within Context {}",
thread_ctx_name
);
// We are shutting down on our own initiative
if let Ok(mut shutdown) = handle.0.shutdown.lock() {
shutdown.clear();
}
panic::resume_unwind(e);
}
}
})
.expect("Failed to spawn Scheduler thread");
let handle = handle_receiver.recv().expect("Context thread init failed");
*handle.0.shutdown.lock().unwrap() = Some(SchedulerShutdown {
context_name,
sender: Some(shutdown_sender),
join: Some(join),
});
handle.set_shutdown(shutdown_sender, join);
handle
}
fn init(context_name: Arc<str>, max_throttling: Duration) -> (tokio::runtime::Runtime, Handle) {
let runtime = tokio::runtime::Builder::new()
.basic_scheduler()
.enable_all()
.max_throttling(max_throttling)
.build()
.expect("Couldn't build the runtime");
fn init(context_name: Arc<str>, max_throttling: Duration) -> Handle {
let handle = CURRENT_SCHEDULER.with(|cur_scheduler| {
let mut cur_scheduler = cur_scheduler.borrow_mut();
if cur_scheduler.is_some() {
panic!("Attempt to initialize an Scheduler on thread where another Scheduler is running.");
}
let scheduler = Arc::new(Scheduler {
context_name: context_name.clone(),
max_throttling,
task_queue: Mutex::new(TaskQueue::new(context_name)),
rt_handle: Mutex::new(runtime.handle().clone()),
shutdown: Mutex::new(None),
let handle = Handle::new(Arc::new(Scheduler {
context_name: context_name.clone(),
max_throttling,
tasks: TaskQueue::new(context_name),
cleanup_ops: ConcurrentQueue::bounded(1000),
must_awake: Mutex::new(false),
must_awake_cvar: Condvar::new(),
}));
*cur_scheduler = Some(handle.downgrade());
handle
});
CURRENT_SCHEDULER.with(|cur_scheduler| {
*cur_scheduler.borrow_mut() = Some(scheduler.downgrade());
});
Reactor::init(handle.max_throttling());
(runtime, scheduler.into())
handle
}
pub fn block_on<F: Future>(future: F) -> <F as Future>::Output {
pub fn block_on<F>(future: F) -> F::Output
where
F: Future + Send + 'static,
F::Output: Send + 'static,
{
assert!(
!Scheduler::is_scheduler_thread(),
"Attempt at blocking on from an existing Scheduler thread."
"Attempt to block within an existing Scheduler thread."
);
let (mut rt, handle) = Scheduler::init(Scheduler::DUMMY_NAME.into(), Duration::ZERO);
let handle = Scheduler::init(Scheduler::DUMMY_NAME.into(), Duration::ZERO);
let this = Arc::clone(&handle.0.scheduler);
let handle_clone = handle.clone();
let task = handle.0.task_queue.lock().unwrap().add(async move {
let (task_id, task) = this.tasks.add(async move {
let res = future.await;
let task_id = TaskId::current().unwrap();
while handle_clone.has_sub_tasks(task_id) {
if handle_clone.drain_sub_tasks(task_id).await.is_err() {
while handle.has_sub_tasks(task_id) {
if handle.drain_sub_tasks(task_id).await.is_err() {
break;
}
}
@ -121,8 +149,7 @@ impl Scheduler {
res
});
let task_id = task.id();
gst_trace!(RUNTIME_CAT, "Blocking on current thread with {:?}", task_id,);
gst_trace!(RUNTIME_CAT, "Blocking on current thread with {:?}", task_id);
let _guard = CallOnDrop::new(|| {
gst_trace!(
@ -130,71 +157,199 @@ impl Scheduler {
"Blocking on current thread with {:?} done",
task_id,
);
handle.remove_task(task_id);
});
rt.block_on(task)
match this.block_on_priv(task) {
Ok(res) => res,
Err(e) => {
gst_error!(
RUNTIME_CAT,
"Panic blocking on Context {}",
&Scheduler::DUMMY_NAME
);
panic::resume_unwind(e);
}
}
}
pub(super) fn is_scheduler_thread() -> bool {
fn block_on_priv<F>(&self, future: F) -> std::thread::Result<F::Output>
where
F: Future + Send + 'static,
F::Output: Send + 'static,
{
let waker = waker_fn(|| ());
let cx = &mut std::task::Context::from_waker(&waker);
pin_mut!(future);
let _guard = CallOnDrop::new(|| Scheduler::close(Arc::clone(&self.context_name)));
let mut last;
loop {
last = Instant::now();
if let Poll::Ready(t) = future.as_mut().poll(cx) {
break Ok(t);
}
Reactor::with_mut(|reactor| {
while let Ok(op) = self.cleanup_ops.pop() {
let _ = reactor.remove_io(&op.0);
}
reactor.react().ok()
});
loop {
match self.tasks.pop_runnable() {
Err(_) => break,
Ok(runnable) => {
panic::catch_unwind(|| runnable.run()).map_err(|err| {
gst_error!(
RUNTIME_CAT,
"A task has panicked within Context {}",
self.context_name
);
err
})?;
}
}
}
let mut must_awake = self.must_awake.lock().unwrap();
let mut must_awake = loop {
if let Some(wait_duration) = self.max_throttling.checked_sub(last.elapsed()) {
let result = self
.must_awake_cvar
.wait_timeout(must_awake, wait_duration)
.unwrap();
must_awake = result.0;
if *must_awake {
break must_awake;
}
} else {
break must_awake;
}
};
*must_awake = false;
}
}
fn wake_up(&self) {
let mut must_awake = self.must_awake.lock().unwrap();
*must_awake = true;
self.must_awake_cvar.notify_one();
}
fn close(context_name: Arc<str>) {
gst_trace!(
RUNTIME_CAT,
"Closing Scheduler for Context {}",
context_name,
);
Reactor::close();
let _ = CURRENT_SCHEDULER.try_with(|cur_scheduler| {
*cur_scheduler.borrow_mut() = None;
});
}
pub fn is_scheduler_thread() -> bool {
CURRENT_SCHEDULER.with(|cur_scheduler| cur_scheduler.borrow().is_some())
}
pub(super) fn current() -> Option<Handle> {
pub fn current() -> Option<Handle> {
CURRENT_SCHEDULER.with(|cur_scheduler| {
cur_scheduler
.borrow()
.as_ref()
.and_then(Weak::upgrade)
.map(Handle::from)
.and_then(HandleWeak::upgrade)
})
}
pub fn is_current(&self) -> bool {
CURRENT_SCHEDULER.with(|cur_scheduler| {
cur_scheduler
.borrow()
.as_ref()
.and_then(HandleWeak::upgrade)
.map_or(false, |cur| {
std::ptr::eq(self, Arc::as_ptr(&cur.0.scheduler))
})
})
}
}
impl Drop for Scheduler {
fn drop(&mut self) {
// No more strong handlers point to this
// Scheduler, so remove its thread local key.
let _ = CURRENT_SCHEDULER.try_with(|cur_scheduler| {
*cur_scheduler.borrow_mut() = None;
});
gst_debug!(
RUNTIME_CAT,
"Terminated: Scheduler for Context '{}'",
"Terminated: Scheduler for Context {}",
self.context_name
);
}
}
#[derive(Debug)]
pub(super) struct SchedulerShutdown {
context_name: Arc<str>,
struct SchedulerShutdown {
scheduler: Arc<Scheduler>,
sender: Option<oneshot::Sender<()>>,
join: Option<thread::JoinHandle<()>>,
}
impl Drop for SchedulerShutdown {
fn drop(&mut self) {
gst_debug!(
RUNTIME_CAT,
"Shutting down Scheduler thread for Context '{}'",
self.context_name
);
self.sender.take().unwrap();
impl SchedulerShutdown {
fn new(scheduler: Arc<Scheduler>) -> Self {
SchedulerShutdown {
scheduler,
sender: None,
join: None,
}
}
gst_trace!(
RUNTIME_CAT,
"Waiting for Scheduler to shutdown for Context '{}'",
self.context_name
);
let _ = self.join.take().unwrap().join();
fn clear(&mut self) {
self.sender = None;
self.join = None;
}
}
impl Drop for SchedulerShutdown {
fn drop(&mut self) {
if let Some(sender) = self.sender.take() {
gst_debug!(
RUNTIME_CAT,
"Shutting down Scheduler thread for Context {}",
self.scheduler.context_name
);
drop(sender);
// Don't block shutting down itself
if !self.scheduler.is_current() {
if let Some(join_handler) = self.join.take() {
gst_trace!(
RUNTIME_CAT,
"Waiting for Scheduler thread to shutdown for Context {}",
self.scheduler.context_name
);
let _ = join_handler.join();
}
}
}
}
}
#[derive(Debug)]
struct HandleInner {
scheduler: Arc<Scheduler>,
shutdown: Mutex<SchedulerShutdown>,
}
#[derive(Clone, Debug)]
pub(super) struct HandleWeak(Weak<Scheduler>);
pub(super) struct HandleWeak(Weak<HandleInner>);
impl HandleWeak {
pub(super) fn upgrade(&self) -> Option<Handle> {
@ -203,97 +358,91 @@ impl HandleWeak {
}
#[derive(Clone, Debug)]
pub(super) struct Handle(Arc<Scheduler>);
pub(super) struct Handle(Arc<HandleInner>);
impl Handle {
fn new(scheduler: Arc<Scheduler>) -> Self {
Handle(Arc::new(HandleInner {
shutdown: Mutex::new(SchedulerShutdown::new(Arc::clone(&scheduler))),
scheduler,
}))
}
fn set_shutdown(&self, sender: oneshot::Sender<()>, join: thread::JoinHandle<()>) {
let mut shutdown = self.0.shutdown.lock().unwrap();
shutdown.sender = Some(sender);
shutdown.join = Some(join);
}
pub fn context_name(&self) -> &str {
&self.0.context_name
&self.0.scheduler.context_name
}
pub fn max_throttling(&self) -> Duration {
self.0.max_throttling
self.0.scheduler.max_throttling
}
pub fn enter<F, R>(&self, f: F) -> R
/// Executes the provided function relatively to this [`Scheduler`]'s [`Reactor`].
///
/// Usefull to initialze i/o sources and timers from outside
/// of a [`Scheduler`].
///
/// # Panic
///
/// This will block current thread and would panic if run
/// from the [`Scheduler`].
pub fn enter<F, O>(&self, f: F) -> O
where
F: FnOnce() -> R,
F: FnOnce() -> O + Send + 'static,
O: Send + 'static,
{
self.0.rt_handle.lock().unwrap().enter(f)
assert!(!self.0.scheduler.is_current());
let task = self.0.scheduler.tasks.add_sync(f);
self.0.scheduler.wake_up();
futures::executor::block_on(task)
}
pub fn add_task<F: Future>(&self, future: F) -> TaskFuture<F> {
let task = self.0.task_queue.lock().unwrap().add(future);
task
}
pub fn remove_task(&self, task_id: TaskId) {
self.0.task_queue.lock().unwrap().remove(task_id);
}
pub fn spawn<F>(&self, future: F, must_awake: bool) -> JoinHandle<F::Output>
pub fn spawn<F>(&self, future: F) -> JoinHandle<F::Output>
where
F: Future + Send + 'static,
F::Output: Send + 'static,
{
let task = self.add_task(future);
let task_id = task.id();
let (sender, receiver) = oneshot::channel();
let (task_id, task) = self.0.scheduler.tasks.add(future);
JoinHandle::new(task_id, task, self)
}
gst_trace!(
RUNTIME_CAT,
"Spawning new task_id {:?} on context {}",
task.id(),
self.0.context_name
);
pub fn spawn_and_awake<F>(&self, future: F) -> JoinHandle<F::Output>
where
F: Future + Send + 'static,
F::Output: Send + 'static,
{
let (task_id, task) = self.0.scheduler.tasks.add(future);
self.0.scheduler.wake_up();
JoinHandle::new(task_id, task, self)
}
let this = self.clone();
let spawn_fut = async move {
gst_trace!(
RUNTIME_CAT,
"Running task_id {:?} on context {}",
task_id,
this.context_name()
);
let _guard = CallOnDrop::new(|| {
gst_trace!(
RUNTIME_CAT,
"Task {:?} on context {} done",
task_id,
this.context_name()
);
this.0.task_queue.lock().unwrap().remove(task_id);
});
let _ = sender.send(task.await);
};
if must_awake {
let _ = self.0.rt_handle.lock().unwrap().awake_and_spawn(spawn_fut);
} else {
let _ = self.0.rt_handle.lock().unwrap().spawn(spawn_fut);
pub fn remove_soure(&self, source: Arc<Source>) {
if self
.0
.scheduler
.cleanup_ops
.push(CleanUpOps(source))
.is_err()
{
gst_warning!(RUNTIME_CAT, "scheduler: cleanup_ops is full");
}
JoinHandle::new(receiver, self, task_id)
}
pub fn has_sub_tasks(&self, task_id: TaskId) -> bool {
let ret = self.0.task_queue.lock().unwrap().has_sub_tasks(task_id);
ret
self.0.scheduler.tasks.has_sub_tasks(task_id)
}
pub fn add_sub_task<T>(&self, task_id: TaskId, sub_task: T) -> Result<(), T>
where
T: Future<Output = SubTaskOutput> + Send + 'static,
{
let res = self
.0
.task_queue
.lock()
.unwrap()
.add_sub_task(task_id, sub_task);
res
self.0.scheduler.tasks.add_sub_task(task_id, sub_task)
}
pub fn downgrade(&self) -> HandleWeak {
@ -301,19 +450,56 @@ impl Handle {
}
pub async fn drain_sub_tasks(&self, task_id: TaskId) -> SubTaskOutput {
let sub_tasks_fut = self.0.task_queue.lock().unwrap().drain_sub_tasks(task_id);
let sub_tasks_fut = self.0.scheduler.tasks.drain_sub_tasks(task_id);
sub_tasks_fut.await
}
}
impl From<Arc<Scheduler>> for Handle {
fn from(arc: Arc<Scheduler>) -> Self {
Handle(arc)
}
}
impl PartialEq for Handle {
fn eq(&self, other: &Self) -> bool {
Arc::ptr_eq(&self.0, &other.0)
}
}
#[cfg(test)]
mod tests {
use super::super::Timer;
use super::*;
#[test]
fn block_on_task_join_handle() {
use std::sync::mpsc;
let (join_sender, join_receiver) = mpsc::channel();
let (shutdown_sender, shutdown_receiver) = oneshot::channel();
std::thread::spawn(move || {
let handle =
Scheduler::init("block_on_task_join_handle".into(), Duration::from_millis(2));
let join_handle = handle.spawn(async {
Timer::after(Duration::from_millis(5)).await;
42
});
let _ = join_sender.send(join_handle);
let _ = handle.0.scheduler.block_on_priv(shutdown_receiver);
});
let task_join_handle = join_receiver.recv().unwrap();
let res = Scheduler::block_on(task_join_handle).unwrap();
let _ = shutdown_sender.send(());
assert_eq!(res, 42);
}
#[test]
fn block_on_timer() {
let res = Scheduler::block_on(async {
Timer::after(Duration::from_millis(5)).await;
42
});
assert_eq!(res, 42);
}
}

View file

@ -1,20 +1,10 @@
// Copyright (C) 2018-2020 Sebastian Dröge <sebastian@centricular.com>
// Copyright (C) 2019-2021 François Laignel <fengalin@free.fr>
//
// This library is free software; you can redistribute it and/or
// modify it under the terms of the GNU Library General Public
// License as published by the Free Software Foundation; either
// version 2 of the License, or (at your option) any later version.
//
// This library is distributed in the hope that it will be useful,
// but WITHOUT ANY WARRANTY; without even the implied warranty of
// MERCHANTABILITY or FITNESS FOR A PARTICULAR PURPOSE. See the GNU
// Library General Public License for more details.
//
// You should have received a copy of the GNU Library General Public
// License along with this library; if not, write to the
// Free Software Foundation, Inc., 51 Franklin Street, Suite 500,
// Boston, MA 02110-1335, USA.
// Take a look at the license at the top of the repository in the LICENSE file.
use async_task::Runnable;
use concurrent_queue::ConcurrentQueue;
use futures::future::BoxFuture;
use futures::prelude::*;
@ -23,13 +13,16 @@ use gst::{gst_log, gst_trace, gst_warning};
use pin_project_lite::pin_project;
use slab::Slab;
use std::cell::Cell;
use std::collections::{HashMap, VecDeque};
use std::collections::VecDeque;
use std::fmt;
use std::pin::Pin;
use std::sync::Arc;
use std::sync::{Arc, Mutex};
use std::task::Poll;
use super::CallOnDrop;
use crate::runtime::RUNTIME_CAT;
thread_local! {
@ -37,15 +30,9 @@ thread_local! {
}
#[derive(Clone, Copy, Eq, PartialEq, Hash, Debug)]
pub struct TaskId(pub(super) u64);
pub struct TaskId(pub(super) usize);
impl TaskId {
const LAST: TaskId = TaskId(u64::MAX);
fn next(task_id: Self) -> Self {
TaskId(task_id.0.wrapping_add(1))
}
pub(super) fn current() -> Option<TaskId> {
CURRENT_TASK_ID.try_with(Cell::get).ok().flatten()
}
@ -62,12 +49,6 @@ pin_project! {
}
impl<F: Future> TaskFuture<F> {
pub fn id(&self) -> TaskId {
self.id
}
}
impl<F: Future> Future for TaskFuture<F> {
type Output = F::Output;
@ -129,57 +110,144 @@ impl fmt::Debug for Task {
#[derive(Debug)]
pub(super) struct TaskQueue {
last_task_id: TaskId,
tasks: HashMap<TaskId, Task>,
runnables: Arc<ConcurrentQueue<Runnable>>,
// FIXME good point about using a slab is that it's probably faster than a HashMap
// However since we reuse the vacant entries, we get the same TaskId
// which can harm debugging. If this is not acceptable, I'll switch back to using
// a HashMap.
tasks: Arc<Mutex<Slab<Task>>>,
context_name: Arc<str>,
}
impl TaskQueue {
pub fn new(context_name: Arc<str>) -> Self {
TaskQueue {
last_task_id: TaskId::LAST,
tasks: HashMap::default(),
runnables: Arc::new(ConcurrentQueue::unbounded()),
tasks: Arc::new(Mutex::new(Slab::new())),
context_name,
}
}
pub fn add<F: Future>(&mut self, future: F) -> TaskFuture<F> {
self.last_task_id = TaskId::next(self.last_task_id);
self.tasks
.insert(self.last_task_id, Task::new(self.last_task_id));
pub fn add<F>(&self, future: F) -> (TaskId, async_task::Task<<F as Future>::Output>)
where
F: Future + Send + 'static,
F::Output: Send + 'static,
{
let tasks_clone = Arc::clone(&self.tasks);
let mut tasks = self.tasks.lock().unwrap();
let task_id = TaskId(tasks.vacant_entry().key());
TaskFuture {
id: self.last_task_id,
future,
}
let context_name = Arc::clone(&self.context_name);
let task_fut = async move {
gst_trace!(
RUNTIME_CAT,
"Running {:?} on context {}",
task_id,
context_name
);
let _guard = CallOnDrop::new(move || {
if let Some(task) = tasks_clone.lock().unwrap().try_remove(task_id.0) {
if !task.sub_tasks.is_empty() {
gst_warning!(
RUNTIME_CAT,
"Task {:?} on context {} has {} pending sub tasks",
task_id,
context_name,
task.sub_tasks.len(),
);
}
}
gst_trace!(
RUNTIME_CAT,
"Done {:?} on context {}",
task_id,
context_name
);
});
TaskFuture {
id: task_id,
future,
}
.await
};
let runnables = Arc::clone(&self.runnables);
let (runnable, task) = async_task::spawn(task_fut, move |runnable| {
runnables.push(runnable).unwrap();
});
tasks.insert(Task::new(task_id));
drop(tasks);
runnable.schedule();
(task_id, task)
}
pub fn remove(&mut self, task_id: TaskId) {
if let Some(task) = self.tasks.remove(&task_id) {
if !task.sub_tasks.is_empty() {
gst_warning!(
pub fn add_sync<F, O>(&self, f: F) -> async_task::Task<O>
where
F: FnOnce() -> O + Send + 'static,
O: Send + 'static,
{
let tasks_clone = Arc::clone(&self.tasks);
let mut tasks = self.tasks.lock().unwrap();
let task_id = TaskId(tasks.vacant_entry().key());
let context_name = Arc::clone(&self.context_name);
let task_fut = async move {
gst_trace!(
RUNTIME_CAT,
"Executing sync function on context {} as {:?}",
context_name,
task_id,
);
let _guard = CallOnDrop::new(move || {
let _ = tasks_clone.lock().unwrap().try_remove(task_id.0);
gst_trace!(
RUNTIME_CAT,
"Task {:?} on context {} has {} pending sub tasks",
"Done executing sync function on context {} as {:?}",
context_name,
task_id,
self.context_name,
task.sub_tasks.len(),
);
}
}
});
f()
};
let runnables = Arc::clone(&self.runnables);
let (runnable, task) = async_task::spawn(task_fut, move |runnable| {
runnables.push(runnable).unwrap();
});
tasks.insert(Task::new(task_id));
drop(tasks);
runnable.schedule();
task
}
pub fn pop_runnable(&self) -> Result<Runnable, concurrent_queue::PopError> {
self.runnables.pop()
}
pub fn has_sub_tasks(&self, task_id: TaskId) -> bool {
self.tasks
.get(&task_id)
.lock()
.unwrap()
.get(task_id.0)
.map(|t| !t.sub_tasks.is_empty())
.unwrap_or(false)
}
pub fn add_sub_task<T>(&mut self, task_id: TaskId, sub_task: T) -> Result<(), T>
pub fn add_sub_task<T>(&self, task_id: TaskId, sub_task: T) -> Result<(), T>
where
T: Future<Output = SubTaskOutput> + Send + 'static,
{
match self.tasks.get_mut(&task_id) {
match self.tasks.lock().unwrap().get_mut(task_id.0) {
Some(task) => {
gst_trace!(
RUNTIME_CAT,
@ -198,12 +266,14 @@ impl TaskQueue {
}
pub fn drain_sub_tasks(
&mut self,
&self,
task_id: TaskId,
) -> impl Future<Output = SubTaskOutput> + Send + 'static {
let sub_tasks = self
.tasks
.get_mut(&task_id)
.lock()
.unwrap()
.get_mut(task_id.0)
.map(|task| (task.drain_sub_tasks(), Arc::clone(&self.context_name)));
async move {

View file

@ -0,0 +1,312 @@
// This is based on https://github.com/smol-rs/async-io
// with adaptations by:
//
// Copyright (C) 2021 François Laignel <fengalin@free.fr>
//
// Take a look at the license at the top of the repository in the LICENSE file.
use futures::stream::Stream;
use std::future::Future;
use std::pin::Pin;
use std::task::{Context, Poll, Waker};
use std::time::{Duration, Instant};
use super::Reactor;
/// A future or stream that emits timed events.
///
/// Timers are futures that output a single [`Instant`] when they fire.
///
/// Timers are also streams that can output [`Instant`]s periodically.
#[derive(Debug)]
pub struct Timer {
/// This timer's ID and last waker that polled it.
///
/// When this field is set to `None`, this timer is not registered in the reactor.
id_and_waker: Option<(usize, Waker)>,
/// The next instant at which this timer fires.
when: Instant,
/// The period.
period: Duration,
}
impl Timer {
/// Creates a timer that emits an event once after the given duration of time.
///
/// When throttling is activated (i.e. when using a non-`0` `wait`
/// duration in `Context::acquire`), timer entries are assigned to
/// the nearest time frame, meaning that the delay might elapse
/// `wait` / 2 ms earlier or later than the expected instant.
///
/// Use [`Timer::after_at_least`] when it's preferable not to return
/// before the expected instant.
pub fn after(duration: Duration) -> Timer {
Timer::at(Instant::now() + duration)
}
/// Creates a timer that emits an event once after the given duration of time.
///
/// See [`Timer::after`] for details. The event won't be emitted before
/// the expected delay has elapsed.
#[track_caller]
pub fn after_at_least(duration: Duration) -> Timer {
Timer::at_least_at(Instant::now() + duration)
}
/// Creates a timer that emits an event once at the given time instant.
///
/// When throttling is activated (i.e. when using a non-`0` `wait`
/// duration in `Context::acquire`), timer entries are assigned to
/// the nearest time frame, meaning that the delay might elapse
/// `wait` / 2 ms earlier or later than the expected instant.
///
/// Use [`Timer::at_least_at`] when it's preferable not to return
/// before the expected instant.
pub fn at(instant: Instant) -> Timer {
Timer::interval_at(instant, Duration::MAX)
}
/// Creates a timer that emits an event once at the given time instant.
///
/// See [`Timer::at`] for details. The event won't be emitted before
/// the expected delay has elapsed.
#[track_caller]
pub fn at_least_at(instant: Instant) -> Timer {
Timer::interval_at_least_at(instant, Duration::MAX)
}
/// Creates a timer that emits events periodically.
pub fn interval(period: Duration) -> Timer {
Timer::interval_at(Instant::now() + period, period)
}
/// Creates a timer that emits events periodically, starting at `start`.
///
/// When throttling is activated (i.e. when using a non-`0` `wait`
/// duration in `Context::acquire`), timer entries are assigned to
/// the nearest time frame, meaning that the delay might elapse
/// `wait` / 2 ms earlier or later than the expected instant.
///
/// Use [`Timer::interval_at_least_at`] when it's preferable not to return
/// before the expected instant.
pub fn interval_at(start: Instant, period: Duration) -> Timer {
Timer {
id_and_waker: None,
when: start,
period,
}
}
/// Creates a timer that emits events periodically, starting at `start`.
///
/// See [`Timer::interval_at`] for details. The event won't be emitted before
/// the expected delay has elapsed.
#[track_caller]
pub fn interval_at_least_at(start: Instant, period: Duration) -> Timer {
Timer {
id_and_waker: None,
when: start + Reactor::with(|reactor| reactor.half_max_throttling()),
period,
}
}
/// Sets the timer to emit an en event once after the given duration of time.
///
/// Note that resetting a timer is different from creating a new timer because
/// [`set_after()`][`Timer::set_after()`] does not remove the waker associated with the task
/// that is polling the timer.
pub fn set_after(&mut self, duration: Duration) {
self.set_at(Instant::now() + duration);
}
/// Sets the timer to emit an event once at the given time instant.
///
/// Note that resetting a timer is different from creating a new timer because
/// [`set_at()`][`Timer::set_at()`] does not remove the waker associated with the task
/// that is polling the timer.
pub fn set_at(&mut self, instant: Instant) {
Reactor::with_mut(|reactor| {
if let Some((id, _)) = self.id_and_waker.as_ref() {
// Deregister the timer from the reactor.
reactor.remove_timer(self.when, *id);
}
// Update the timeout.
self.when = instant;
if let Some((id, waker)) = self.id_and_waker.as_mut() {
// Re-register the timer with the new timeout.
*id = reactor.insert_timer(self.when, waker);
}
})
}
/// Sets the timer to emit events periodically.
///
/// Note that resetting a timer is different from creating a new timer because
/// [`set_interval()`][`Timer::set_interval()`] does not remove the waker associated with the
/// task that is polling the timer.
pub fn set_interval(&mut self, period: Duration) {
self.set_interval_at(Instant::now() + period, period);
}
/// Sets the timer to emit events periodically, starting at `start`.
///
/// Note that resetting a timer is different from creating a new timer because
/// [`set_interval_at()`][`Timer::set_interval_at()`] does not remove the waker associated with
/// the task that is polling the timer.
pub fn set_interval_at(&mut self, start: Instant, period: Duration) {
// Note: the timer might have been registered on an Executor and then transfered to another.
Reactor::with_mut(|reactor| {
if let Some((id, _)) = self.id_and_waker.as_ref() {
// Deregister the timer from the reactor.
reactor.remove_timer(self.when, *id);
}
self.when = start;
self.period = period;
if let Some((id, waker)) = self.id_and_waker.as_mut() {
// Re-register the timer with the new timeout.
*id = reactor.insert_timer(self.when, waker);
}
})
}
}
impl Drop for Timer {
fn drop(&mut self) {
if let Some((id, _)) = self.id_and_waker.take() {
Reactor::with_mut(|reactor| {
reactor.remove_timer(self.when, id);
});
}
}
}
impl Future for Timer {
type Output = ();
fn poll(self: Pin<&mut Self>, cx: &mut Context<'_>) -> Poll<Self::Output> {
match self.poll_next(cx) {
Poll::Ready(Some(_)) => Poll::Ready(()),
Poll::Pending => Poll::Pending,
Poll::Ready(None) => unreachable!(),
}
}
}
impl Stream for Timer {
type Item = ();
fn poll_next(mut self: Pin<&mut Self>, cx: &mut Context<'_>) -> Poll<Option<Self::Item>> {
Reactor::with_mut(|reactor| {
if Instant::now() + reactor.half_max_throttling() >= self.when {
if let Some((id, _)) = self.id_and_waker.take() {
// Deregister the timer from the reactor.
reactor.remove_timer(self.when, id);
}
let when = self.when;
if let Some(next) = when.checked_add(self.period) {
self.when = next;
// Register the timer in the reactor.
let id = reactor.insert_timer(self.when, cx.waker());
self.id_and_waker = Some((id, cx.waker().clone()));
}
Poll::Ready(Some(()))
} else {
match &self.id_and_waker {
None => {
// Register the timer in the reactor.
let id = reactor.insert_timer(self.when, cx.waker());
self.id_and_waker = Some((id, cx.waker().clone()));
}
Some((id, w)) if !w.will_wake(cx.waker()) => {
// Deregister the timer from the reactor to remove the old waker.
reactor.remove_timer(self.when, *id);
// Register the timer in the reactor with the new waker.
let id = reactor.insert_timer(self.when, cx.waker());
self.id_and_waker = Some((id, cx.waker().clone()));
}
Some(_) => {}
}
Poll::Pending
}
})
}
}
#[cfg(test)]
mod tests {
use std::time::{Duration, Instant};
use super::Timer;
use crate::runtime::executor::Scheduler;
const MAX_THROTTLING: Duration = Duration::from_millis(10);
const DELAY: Duration = Duration::from_millis(12);
#[test]
fn delay_for() {
gst::init().unwrap();
let handle = Scheduler::start("delay_for", MAX_THROTTLING);
let elapsed = futures::executor::block_on(handle.spawn(async {
let now = Instant::now();
Timer::after(DELAY).await;
now.elapsed()
}))
.unwrap();
// Due to throttling, timer may be fired earlier
assert!(elapsed + MAX_THROTTLING / 2 >= DELAY);
}
#[test]
fn delay_for_at_least() {
gst::init().unwrap();
let handle = Scheduler::start("delay_for_at_least", MAX_THROTTLING);
let elapsed = futures::executor::block_on(handle.spawn(async {
let now = Instant::now();
Timer::after_at_least(DELAY).await;
now.elapsed()
}))
.unwrap();
// Never returns earlier than DELAY
assert!(elapsed >= DELAY);
}
#[test]
fn interval() {
use futures::prelude::*;
gst::init().unwrap();
let handle = Scheduler::start("interval", MAX_THROTTLING);
let join_handle = handle.spawn(async move {
let start = Instant::now();
let mut interval = Timer::interval(DELAY);
interval.next().await;
// Due to throttling, timer may be fired earlier
assert!(start.elapsed() + MAX_THROTTLING / 2 >= DELAY);
interval.next().await;
// Due to throttling, timer may be fired earlier
assert!(start.elapsed() + MAX_THROTTLING >= 2 * DELAY);
});
futures::executor::block_on(join_handle).unwrap();
}
}

View file

@ -1,19 +1,6 @@
// Copyright (C) 2019 François Laignel <fengalin@free.fr>
// Copyright (C) 2019-2021 François Laignel <fengalin@free.fr>
//
// This library is free software; you can redistribute it and/or
// modify it under the terms of the GNU Library General Public
// License as published by the Free Software Foundation; either
// version 2 of the License, or (at your option) any later version.
//
// This library is distributed in the hope that it will be useful,
// but WITHOUT ANY WARRANTY; without even the implied warranty of
// MERCHANTABILITY or FITNESS FOR A PARTICULAR PURPOSE. See the GNU
// Library General Public License for more details.
//
// You should have received a copy of the GNU Library General Public
// License along with this library; if not, write to the
// Free Software Foundation, Inc., 51 Franklin Street, Suite 500,
// Boston, MA 02110-1335, USA.
// Take a look at the license at the top of the repository in the LICENSE file.
//! A `runtime` for the `threadshare` GStreamer plugins framework.
//!
@ -31,8 +18,7 @@
//! See this [talk] ([slides]) for a presentation of the motivations and principles,
//! and this [blog post].
//!
//! FIXME change this.
//! Current implementation uses the crate [`tokio`].
//! Current implementation uses a custom executor mostly based on the [`smol`] ecosystem.
//!
//! Most `Element`s implementations should use the high-level features provided by [`PadSrc`] &
//! [`PadSink`].
@ -40,12 +26,12 @@
//! [talk]: https://gstconf.ubicast.tv/videos/when-adding-more-threads-adds-more-problems-thread-sharing-between-elements-in-gstreamer/
//! [slides]: https://gstreamer.freedesktop.org/data/events/gstreamer-conference/2018/Sebastian%20Dr%C3%B6ge%20-%20When%20adding%20more%20threads%20adds%20more%20problems:%20Thread-sharing%20between%20elements%20in%20GStreamer.pdf
//! [blog post]: https://coaxion.net/blog/2018/04/improving-gstreamer-performance-on-a-high-number-of-network-streams-by-sharing-threads-between-elements-with-rusts-tokio-crate
//! [`tokio`]: https://crates.io/crates/tokio
//! [`smol`]: https://github.com/smol-rs/
//! [`PadSrc`]: pad/struct.PadSrc.html
//! [`PadSink`]: pad/struct.PadSink.html
pub mod executor;
pub use executor::{Context, JoinHandle, SubTaskOutput};
pub use executor::{Async, Context, JoinHandle, SubTaskOutput, Timer};
pub mod pad;
pub use pad::{PadSink, PadSinkRef, PadSinkWeak, PadSrc, PadSrcRef, PadSrcWeak};
@ -59,6 +45,7 @@ pub mod prelude {
}
pub mod time;
pub use time::{delay_for, delay_for_at_least};
use once_cell::sync::Lazy;

View file

@ -729,13 +729,13 @@ macro_rules! exec_action {
let join_handle = {
let mut task_inner = $task_inner.lock().unwrap();
let join_handle = $context.awake_and_spawn(action_fut);
let join_handle = $context.spawn_and_awake(action_fut);
task_inner.spawned_task_id = Some(join_handle.task_id());
join_handle
};
let (this, res) = join_handle.map(|res| res.unwrap()).await;
let (this, res) = join_handle.await.unwrap();
$self = this;
match res {
@ -989,7 +989,7 @@ impl StateMachine {
// Unprepare is not joined by an ack_rx but by joining the state machine
// handle, so we don't need to keep track of the spwaned_task_id
context
.awake_and_spawn(async move {
.spawn_and_awake(async move {
self.task_impl.unprepare().await;
while Context::current_has_sub_tasks() {
@ -1094,13 +1094,13 @@ impl StateMachine {
let join_handle = {
let mut task_inner = task_inner.lock().unwrap();
let join_handle = context.awake_and_spawn(loop_fut);
let join_handle = context.spawn_and_awake(loop_fut);
task_inner.spawned_task_id = Some(join_handle.task_id());
join_handle
};
join_handle.map(|res| res.unwrap()).await
join_handle.await.unwrap()
}
async fn run_loop(&mut self, task_inner: Arc<Mutex<TaskInner>>) -> Result<(), gst::FlowError> {
@ -1601,7 +1601,7 @@ mod tests {
let task = Task::default();
let (mut prepare_sender, prepare_receiver) = mpsc::channel(1);
task.prepare(TaskPrepareTest { prepare_receiver }, context.clone())
task.prepare(TaskPrepareTest { prepare_receiver }, context)
.unwrap();
let start_ctx = Context::acquire("prepare_start_ok_requester", Duration::ZERO).unwrap();
@ -2264,7 +2264,7 @@ mod tests {
async move {
gst_debug!(RUNTIME_CAT, "pause_from_loop: entering iteration");
tokio::time::delay_for(Duration::from_millis(50)).await;
crate::runtime::time::delay_for(Duration::from_millis(50)).await;
gst_debug!(RUNTIME_CAT, "pause_from_loop: pause from iteration");
assert_eq!(
@ -2715,14 +2715,14 @@ mod tests {
gst::init().unwrap();
struct TaskTimerTest {
timer: Option<tokio::time::Delay>,
timer: Option<crate::runtime::Timer>,
timer_elapsed_sender: Option<oneshot::Sender<()>>,
}
impl TaskImpl for TaskTimerTest {
fn start(&mut self) -> BoxFuture<'_, Result<(), gst::ErrorMessage>> {
async move {
self.timer = Some(tokio::time::delay_for(Duration::from_millis(50)));
self.timer = Some(crate::runtime::time::delay_for(Duration::from_millis(50)));
gst_debug!(RUNTIME_CAT, "start_timer: started");
Ok(())
}

View file

@ -1,28 +1,12 @@
// Copyright (C) 2020 François Laignel <fengalin@free.fr>
// Copyright (C) 2020-2021 François Laignel <fengalin@free.fr>
//
// This library is free software; you can redistribute it and/or
// modify it under the terms of the GNU Library General Public
// License as published by the Free Software Foundation; either
// version 2 of the License, or (at your option) any later version.
//
// This library is distributed in the hope that it will be useful,
// but WITHOUT ANY WARRANTY; without even the implied warranty of
// MERCHANTABILITY or FITNESS FOR A PARTICULAR PURPOSE. See the GNU
// Library General Public License for more details.
//
// You should have received a copy of the GNU Library General Public
// License along with this library; if not, write to the
// Free Software Foundation, Inc., 51 Franklin Street, Suite 500,
// Boston, MA 02110-1335, USA.
// Take a look at the license at the top of the repository in the LICENSE file.
//! Wrappers for the underlying runtime specific time related Futures.
use futures::prelude::*;
use futures::stream::StreamExt;
use std::time::Duration;
use super::Context;
use super::executor::Timer;
/// Wait until the given `delay` has elapsed.
///
@ -35,10 +19,8 @@ use super::Context;
///
/// Use [`delay_for_at_least`] when it's preferable not to return
/// before the expected instant.
pub async fn delay_for(delay: Duration) {
if delay > Duration::ZERO {
tokio::time::delay_for(delay).map(drop).await;
}
pub fn delay_for(delay: Duration) -> Timer {
Timer::after(delay)
}
/// Wait until at least the given `delay` has elapsed.
@ -47,63 +29,14 @@ pub async fn delay_for(delay: Duration) {
///
/// See [`delay_for`] for details. This method won't return before
/// the expected delay has elapsed.
pub async fn delay_for_at_least(delay: Duration) {
if delay > Duration::ZERO {
tokio::time::delay_for(
delay + Context::current().map_or(Duration::ZERO, |ctx| ctx.wait_duration() / 2),
)
.map(drop)
.await;
}
#[track_caller]
pub fn delay_for_at_least(delay: Duration) -> Timer {
Timer::after_at_least(delay)
}
/// Builds a `Stream` that yields at `interval.
/// Builds a `Stream` that yields at `interval`.
///
/// This must be called from within the target runtime environment.
pub fn interval(interval: Duration) -> impl Stream<Item = ()> {
tokio::time::interval(interval).map(drop)
}
#[cfg(test)]
mod tests {
use std::time::{Duration, Instant};
use crate::runtime::{executor, Context};
const MAX_THROTTLING: Duration = Duration::from_millis(10);
const DELAY: Duration = Duration::from_millis(12);
#[test]
fn delay_for() {
gst::init().unwrap();
let context = Context::acquire("delay_for", MAX_THROTTLING).unwrap();
let elapsed = executor::block_on(context.spawn(async {
let now = Instant::now();
crate::runtime::time::delay_for(DELAY).await;
now.elapsed()
}))
.unwrap();
// Due to throttling, timer may be fired earlier
assert!(elapsed + MAX_THROTTLING / 2 >= DELAY);
}
#[test]
fn delay_for_at_least() {
gst::init().unwrap();
let context = Context::acquire("delay_for_at_least", MAX_THROTTLING).unwrap();
let elapsed = executor::block_on(context.spawn(async {
let now = Instant::now();
crate::runtime::time::delay_for_at_least(DELAY).await;
now.elapsed()
}))
.unwrap();
// Never returns earlier that DELAY
assert!(elapsed >= DELAY);
}
pub fn interval(interval: Duration) -> Timer {
Timer::interval(interval)
}

View file

@ -24,12 +24,14 @@ use gst::{gst_debug, gst_error, gst_log};
use once_cell::sync::Lazy;
use std::io;
use gio::prelude::*;
use std::error;
use std::fmt;
use std::io;
use std::net::UdpSocket;
use crate::runtime::Async;
#[cfg(unix)]
use std::os::unix::io::{AsRawFd, FromRawFd, IntoRawFd, RawFd};
@ -302,7 +304,7 @@ unsafe fn dup_socket(socket: usize) -> usize {
socket
}
pub fn wrap_socket(socket: &tokio::net::UdpSocket) -> Result<GioSocketWrapper, gst::ErrorMessage> {
pub fn wrap_socket(socket: &Async<UdpSocket>) -> Result<GioSocketWrapper, gst::ErrorMessage> {
#[cfg(unix)]
unsafe {
let fd = libc::dup(socket.as_raw_fd());
@ -328,9 +330,7 @@ pub fn wrap_socket(socket: &tokio::net::UdpSocket) -> Result<GioSocketWrapper, g
}
#[cfg(windows)]
unsafe {
// FIXME: Needs https://github.com/tokio-rs/tokio/pull/806
// and https://github.com/carllerche/mio/pull/859
let fd = unreachable!(); //dup_socket(socket.as_raw_socket() as _) as _;
let fd = socket.as_raw_socket();
// This is unsafe because it allows us to share the fd between the socket and the
// GIO socket below, but safety of this is the job of the application

View file

@ -28,19 +28,18 @@ use gst::{gst_debug, gst_error, gst_log, gst_trace};
use once_cell::sync::Lazy;
use std::io;
use std::net::{IpAddr, SocketAddr};
use std::net::{IpAddr, SocketAddr, TcpStream};
use std::sync::Arc;
use std::sync::Mutex as StdMutex;
use std::time::Duration;
use std::u16;
use std::u32;
use tokio::io::AsyncReadExt;
use crate::runtime::prelude::*;
use crate::runtime::task;
use crate::runtime::{Context, PadSrc, PadSrcRef, PadSrcWeak, Task, TaskState};
use crate::runtime::Async;
use crate::socket::{Socket, SocketError, SocketRead};
const DEFAULT_HOST: Option<&str> = Some("127.0.0.1");
@ -73,10 +72,10 @@ impl Default for Settings {
}
}
struct TcpClientReader(tokio::net::TcpStream);
struct TcpClientReader(Async<TcpStream>);
impl TcpClientReader {
pub fn new(socket: tokio::net::TcpStream) -> Self {
pub fn new(socket: Async<TcpStream>) -> Self {
TcpClientReader(socket)
}
}
@ -294,7 +293,7 @@ impl TaskImpl for TcpClientSrcTask {
async move {
gst_log!(CAT, obj: &self.element, "Preparing task connecting to {:?}", self.saddr);
let socket = tokio::net::TcpStream::connect(self.saddr)
let socket = Async::<TcpStream>::connect(self.saddr)
.await
.map_err(|err| {
gst::error_msg!(
@ -647,7 +646,7 @@ impl ObjectImpl for TcpClientSrc {
settings.context = value
.get::<Option<String>>()
.expect("type checked upstream")
.unwrap_or_else(|| "".into());
.unwrap_or_else(|| DEFAULT_CONTEXT.into());
}
"context-wait" => {
settings.context_wait = Duration::from_millis(

View file

@ -31,11 +31,11 @@ use gst::{
use once_cell::sync::Lazy;
use crate::runtime::prelude::*;
use crate::runtime::{self, Context, PadSink, PadSinkRef, Task};
use crate::runtime::{self, Async, Context, PadSink, PadSinkRef, Task};
use crate::socket::{wrap_socket, GioSocketWrapper};
use std::mem;
use std::net::{IpAddr, Ipv4Addr, SocketAddr};
use std::net::{IpAddr, Ipv4Addr, SocketAddr, UdpSocket};
use std::sync::Mutex as StdMutex;
use std::sync::{Arc, RwLock};
use std::time::Duration;
@ -124,8 +124,8 @@ struct UdpSinkPadHandlerInner {
sync: bool,
segment: Option<gst::Segment>,
latency: Option<gst::ClockTime>,
socket: Arc<Mutex<Option<tokio::net::UdpSocket>>>,
socket_v6: Arc<Mutex<Option<tokio::net::UdpSocket>>>,
socket: Arc<Mutex<Option<Async<UdpSocket>>>>,
socket_v6: Arc<Mutex<Option<Async<UdpSocket>>>>,
#[allow(clippy::rc_buffer)]
clients: Arc<Vec<SocketAddr>>,
clients_to_configure: Vec<SocketAddr>,
@ -202,8 +202,8 @@ impl UdpSinkPadHandlerInner {
#[derive(Debug)]
enum SocketQualified {
Ipv4(tokio::net::UdpSocket),
Ipv6(tokio::net::UdpSocket),
Ipv4(Async<UdpSocket>),
Ipv6(Async<UdpSocket>),
}
#[derive(Clone, Debug)]
@ -259,8 +259,8 @@ impl UdpSinkPadHandler {
fn configure_client(
&self,
settings: &Settings,
socket: &mut Option<tokio::net::UdpSocket>,
socket_v6: &mut Option<tokio::net::UdpSocket>,
socket: &mut Option<Async<UdpSocket>>,
socket_v6: &mut Option<Async<UdpSocket>>,
client: &SocketAddr,
) -> Result<(), gst::ErrorMessage> {
if client.ip().is_multicast() {
@ -269,7 +269,8 @@ impl UdpSinkPadHandler {
if let Some(socket) = socket.as_mut() {
if settings.auto_multicast {
socket
.join_multicast_v4(addr, Ipv4Addr::new(0, 0, 0, 0))
.as_ref()
.join_multicast_v4(&addr, &Ipv4Addr::new(0, 0, 0, 0))
.map_err(|err| {
error_msg!(
gst::ResourceError::OpenWrite,
@ -278,7 +279,7 @@ impl UdpSinkPadHandler {
})?;
}
if settings.multicast_loop {
socket.set_multicast_loop_v4(true).map_err(|err| {
socket.as_ref().set_multicast_loop_v4(true).map_err(|err| {
error_msg!(
gst::ResourceError::OpenWrite,
["Failed to set multicast loop: {}", err]
@ -286,6 +287,7 @@ impl UdpSinkPadHandler {
})?;
}
socket
.as_ref()
.set_multicast_ttl_v4(settings.ttl_mc)
.map_err(|err| {
error_msg!(
@ -298,7 +300,7 @@ impl UdpSinkPadHandler {
IpAddr::V6(addr) => {
if let Some(socket) = socket_v6.as_mut() {
if settings.auto_multicast {
socket.join_multicast_v6(&addr, 0).map_err(|err| {
socket.as_ref().join_multicast_v6(&addr, 0).map_err(|err| {
error_msg!(
gst::ResourceError::OpenWrite,
["Failed to join multicast group: {}", err]
@ -306,7 +308,7 @@ impl UdpSinkPadHandler {
})?;
}
if settings.multicast_loop {
socket.set_multicast_loop_v6(true).map_err(|err| {
socket.as_ref().set_multicast_loop_v6(true).map_err(|err| {
error_msg!(
gst::ResourceError::OpenWrite,
["Failed to set multicast loop: {}", err]
@ -321,7 +323,7 @@ impl UdpSinkPadHandler {
match client.ip() {
IpAddr::V4(_) => {
if let Some(socket) = socket.as_mut() {
socket.set_ttl(settings.ttl).map_err(|err| {
socket.as_ref().set_ttl(settings.ttl).map_err(|err| {
error_msg!(
gst::ResourceError::OpenWrite,
["Failed to set unicast ttl: {}", err]
@ -331,7 +333,7 @@ impl UdpSinkPadHandler {
}
IpAddr::V6(_) => {
if let Some(socket) = socket_v6.as_mut() {
socket.set_ttl(settings.ttl).map_err(|err| {
socket.as_ref().set_ttl(settings.ttl).map_err(|err| {
error_msg!(
gst::ResourceError::OpenWrite,
["Failed to set unicast ttl: {}", err]
@ -348,8 +350,8 @@ impl UdpSinkPadHandler {
fn unconfigure_client(
&self,
settings: &Settings,
socket: &mut Option<tokio::net::UdpSocket>,
socket_v6: &mut Option<tokio::net::UdpSocket>,
socket: &mut Option<Async<UdpSocket>>,
socket_v6: &mut Option<Async<UdpSocket>>,
client: &SocketAddr,
) -> Result<(), gst::ErrorMessage> {
if client.ip().is_multicast() {
@ -358,7 +360,8 @@ impl UdpSinkPadHandler {
if let Some(socket) = socket.as_mut() {
if settings.auto_multicast {
socket
.leave_multicast_v4(addr, Ipv4Addr::new(0, 0, 0, 0))
.as_ref()
.leave_multicast_v4(&addr, &Ipv4Addr::new(0, 0, 0, 0))
.map_err(|err| {
error_msg!(
gst::ResourceError::OpenWrite,
@ -371,12 +374,15 @@ impl UdpSinkPadHandler {
IpAddr::V6(addr) => {
if let Some(socket) = socket_v6.as_mut() {
if settings.auto_multicast {
socket.leave_multicast_v6(&addr, 0).map_err(|err| {
error_msg!(
gst::ResourceError::OpenWrite,
["Failed to join multicast group: {}", err]
)
})?;
socket
.as_ref()
.leave_multicast_v6(&addr, 0)
.map_err(|err| {
error_msg!(
gst::ResourceError::OpenWrite,
["Failed to join multicast group: {}", err]
)
})?;
}
}
}
@ -485,7 +491,7 @@ impl UdpSinkPadHandler {
if let Some(socket) = socket.as_mut() {
gst_log!(CAT, obj: element, "Sending to {:?}", &client);
socket.send_to(&data, client).await.map_err(|err| {
socket.send_to(&data, *client).await.map_err(|err| {
element_error!(
element,
gst::StreamError::Failed,
@ -524,7 +530,9 @@ impl UdpSinkPadHandler {
let now = element.current_running_time();
match running_time.into().opt_checked_sub(now) {
Ok(Some(delay)) => runtime::time::delay_for(delay.into()).await,
Ok(Some(delay)) => {
let _ = runtime::time::delay_for(delay.into()).await;
}
_ => runtime::executor::yield_now().await,
}
}
@ -731,13 +739,13 @@ impl UdpSink {
let socket_qualified: SocketQualified;
if let Some(ref wrapped_socket) = wrapped_socket {
let socket = wrapped_socket.get();
let socket: UdpSocket = wrapped_socket.get();
let socket = context.enter(|| {
tokio::net::UdpSocket::from_std(socket).map_err(|err| {
Async::<UdpSocket>::try_from(socket).map_err(|err| {
error_msg!(
gst::ResourceError::OpenWrite,
["Failed to setup socket for tokio: {}", err]
["Failed to setup Async socket: {}", err]
)
})
})?;
@ -811,10 +819,10 @@ impl UdpSink {
})?;
let socket = context.enter(|| {
tokio::net::UdpSocket::from_std(socket.into()).map_err(|err| {
Async::<UdpSocket>::try_from(socket).map_err(|err| {
error_msg!(
gst::ResourceError::OpenWrite,
["Failed to setup socket for tokio: {}", err]
["Failed to setup Async socket: {}", err]
)
})
})?;
@ -1253,7 +1261,7 @@ impl ObjectImpl for UdpSink {
settings.context = value
.get::<Option<String>>()
.expect("type checked upstream")
.unwrap_or_else(|| "".into());
.unwrap_or_else(|| DEFAULT_CONTEXT.into());
}
"context-wait" => {
settings.context_wait = Duration::from_millis(

View file

@ -29,14 +29,14 @@ use once_cell::sync::Lazy;
use std::i32;
use std::io;
use std::net::{IpAddr, Ipv4Addr, Ipv6Addr, SocketAddr};
use std::net::{IpAddr, Ipv4Addr, Ipv6Addr, SocketAddr, UdpSocket};
use std::sync::Arc;
use std::sync::Mutex as StdMutex;
use std::time::Duration;
use std::u16;
use crate::runtime::prelude::*;
use crate::runtime::{Context, PadSrc, PadSrcRef, PadSrcWeak, Task};
use crate::runtime::{Async, Context, PadSrc, PadSrcRef, PadSrcWeak, Task};
use crate::socket::{wrap_socket, GioSocketWrapper, Socket, SocketError, SocketRead};
@ -83,10 +83,10 @@ impl Default for Settings {
}
#[derive(Debug)]
struct UdpReader(tokio::net::UdpSocket);
struct UdpReader(Async<UdpSocket>);
impl UdpReader {
fn new(socket: tokio::net::UdpSocket) -> Self {
fn new(socket: Async<UdpSocket>) -> Self {
UdpReader(socket)
}
}
@ -438,8 +438,6 @@ impl UdpSrc {
})?;
let socket = if let Some(ref wrapped_socket) = settings_guard.socket {
use std::net::UdpSocket;
let socket: UdpSocket;
#[cfg(unix)]
@ -452,10 +450,10 @@ impl UdpSrc {
}
let socket = context.enter(|| {
tokio::net::UdpSocket::from_std(socket).map_err(|err| {
Async::<UdpSocket>::try_from(socket).map_err(|err| {
gst::error_msg!(
gst::ResourceError::OpenRead,
["Failed to setup socket for tokio: {}", err]
["Failed to setup Async socket: {}", err]
)
})
})?;
@ -555,10 +553,10 @@ impl UdpSrc {
})?;
let socket = context.enter(|| {
tokio::net::UdpSocket::from_std(socket.into()).map_err(|err| {
Async::<UdpSocket>::try_from(socket).map_err(|err| {
gst::error_msg!(
gst::ResourceError::OpenRead,
["Failed to setup socket for tokio: {}", err]
["Failed to setup Async socket: {}", err]
)
})
})?;
@ -568,7 +566,8 @@ impl UdpSrc {
match addr {
IpAddr::V4(addr) => {
socket
.join_multicast_v4(addr, Ipv4Addr::new(0, 0, 0, 0))
.as_ref()
.join_multicast_v4(&addr, &Ipv4Addr::new(0, 0, 0, 0))
.map_err(|err| {
gst::error_msg!(
gst::ResourceError::OpenRead,
@ -577,7 +576,7 @@ impl UdpSrc {
})?;
}
IpAddr::V6(addr) => {
socket.join_multicast_v6(&addr, 0).map_err(|err| {
socket.as_ref().join_multicast_v6(&addr, 0).map_err(|err| {
gst::error_msg!(
gst::ResourceError::OpenRead,
["Failed to join multicast group: {}", err]
@ -592,7 +591,7 @@ impl UdpSrc {
socket
};
let port: i32 = socket.local_addr().unwrap().port().into();
let port: i32 = socket.as_ref().local_addr().unwrap().port().into();
let settings = if settings_guard.port != port {
settings_guard.port = port;
let settings = settings_guard.clone();
@ -834,7 +833,7 @@ impl ObjectImpl for UdpSrc {
settings.context = value
.get::<Option<String>>()
.expect("type checked upstream")
.unwrap_or_else(|| "".into());
.unwrap_or_else(|| DEFAULT_CONTEXT.into());
}
"context-wait" => {
settings.context_wait = Duration::from_millis(

View file

@ -35,8 +35,8 @@ fn test_push() {
let pipeline = gst::Pipeline::new(None);
let fakesrc = gst::ElementFactory::make("fakesrc", None).unwrap();
let proxysink = gst::ElementFactory::make("ts-proxysink", None).unwrap();
let proxysrc = gst::ElementFactory::make("ts-proxysrc", None).unwrap();
let proxysink = gst::ElementFactory::make("ts-proxysink", Some("proxysink::test1")).unwrap();
let proxysrc = gst::ElementFactory::make("ts-proxysrc", Some("proxysrc::test1")).unwrap();
let appsink = gst::ElementFactory::make("appsink", None).unwrap();
pipeline
@ -46,8 +46,9 @@ fn test_push() {
proxysrc.link(&appsink).unwrap();
fakesrc.set_property("num-buffers", 3i32);
proxysink.set_property("proxy-context", "test1");
proxysrc.set_property("proxy-context", "test1");
proxysink.set_property("proxy-context", "proxy::test1_proxy");
proxysrc.set_property("proxy-context", "proxy::test1_proxy");
proxysrc.set_property("context", "proxy::test");
appsink.set_property("emit-signals", true);
@ -100,10 +101,10 @@ fn test_from_pipeline_to_pipeline() {
let pipe_1 = gst::Pipeline::new(None);
let fakesrc = gst::ElementFactory::make("fakesrc", None).unwrap();
let pxsink = gst::ElementFactory::make("ts-proxysink", None).unwrap();
let pxsink = gst::ElementFactory::make("ts-proxysink", Some("proxysink::test2")).unwrap();
let pipe_2 = gst::Pipeline::new(None);
let pxsrc = gst::ElementFactory::make("ts-proxysrc", None).unwrap();
let pxsrc = gst::ElementFactory::make("ts-proxysrc", Some("proxysrc::test2")).unwrap();
let fakesink = gst::ElementFactory::make("fakesink", None).unwrap();
pipe_1.add_many(&[&fakesrc, &pxsink]).unwrap();
@ -112,8 +113,9 @@ fn test_from_pipeline_to_pipeline() {
pipe_2.add_many(&[&pxsrc, &fakesink]).unwrap();
pxsrc.link(&fakesink).unwrap();
pxsink.set_property("proxy-context", "test2");
pxsrc.set_property("proxy-context", "test2");
pxsink.set_property("proxy-context", "proxy::test2_proxy");
pxsrc.set_property("proxy-context", "proxy::test2_proxy");
pxsrc.set_property("context", "proxy::test");
pipe_1.set_state(gst::State::Paused).unwrap();
pipe_2.set_state(gst::State::Paused).unwrap();
@ -131,12 +133,12 @@ fn test_from_pipeline_to_pipeline_and_back() {
init();
let pipe_1 = gst::Pipeline::new(None);
let pxsrc_1 = gst::ElementFactory::make("ts-proxysrc", None).unwrap();
let pxsink_1 = gst::ElementFactory::make("ts-proxysink", None).unwrap();
let pxsrc_1 = gst::ElementFactory::make("ts-proxysrc", Some("proxysrc1::test3")).unwrap();
let pxsink_1 = gst::ElementFactory::make("ts-proxysink", Some("proxysink1::test3")).unwrap();
let pipe_2 = gst::Pipeline::new(None);
let pxsrc_2 = gst::ElementFactory::make("ts-proxysrc", None).unwrap();
let pxsink_2 = gst::ElementFactory::make("ts-proxysink", None).unwrap();
let pxsrc_2 = gst::ElementFactory::make("ts-proxysrc", Some("proxysrc2::test3")).unwrap();
let pxsink_2 = gst::ElementFactory::make("ts-proxysink", Some("proxysink2::test3")).unwrap();
pipe_1.add_many(&[&pxsrc_1, &pxsink_1]).unwrap();
pxsrc_1.link(&pxsink_1).unwrap();
@ -144,11 +146,13 @@ fn test_from_pipeline_to_pipeline_and_back() {
pipe_2.add_many(&[&pxsrc_2, &pxsink_2]).unwrap();
pxsrc_2.link(&pxsink_2).unwrap();
pxsrc_1.set_property("proxy-context", "test3");
pxsink_2.set_property("proxy-context", "test3");
pxsrc_1.set_property("proxy-context", "proxy::test3_proxy1");
pxsrc_1.set_property("context", "proxy::test");
pxsink_2.set_property("proxy-context", "proxy::test3_proxy1");
pxsrc_2.set_property("proxy-context", "test4");
pxsink_1.set_property("proxy-context", "test4");
pxsrc_2.set_property("proxy-context", "proxy::test3_proxy2");
pxsrc_2.set_property("context", "proxy::test");
pxsink_1.set_property("proxy-context", "proxy::test3_proxy2");
pipe_1.set_state(gst::State::Paused).unwrap();
pipe_2.set_state(gst::State::Paused).unwrap();