quinn_proto/connection/streams/mod.rs
1use std::{
2 collections::{BinaryHeap, hash_map},
3 io,
4};
5
6use bytes::Bytes;
7use thiserror::Error;
8use tracing::trace;
9
10use super::spaces::{Retransmits, ThinRetransmits};
11use crate::{
12 Dir, StreamId, VarInt,
13 connection::streams::state::{get_or_insert_recv, get_or_insert_send},
14 frame,
15};
16
17mod recv;
18use recv::Recv;
19pub use recv::{Chunks, ReadError, ReadableError};
20
21mod send;
22pub(crate) use send::{ByteSlice, BytesArray};
23pub use send::{BytesSource, FinishError, WriteError, Written};
24use send::{Send, SendState};
25
26mod state;
27#[allow(unreachable_pub)] // fuzzing only
28pub use state::StreamsState;
29
30/// Access to streams
31pub struct Streams<'a> {
32 pub(super) state: &'a mut StreamsState,
33 pub(super) conn_state: &'a super::State,
34}
35
36#[allow(clippy::needless_lifetimes)] // Needed for cfg(fuzzing)
37impl<'a> Streams<'a> {
38 #[cfg(fuzzing)]
39 pub fn new(state: &'a mut StreamsState, conn_state: &'a super::State) -> Self {
40 Self { state, conn_state }
41 }
42
43 /// Open a single stream if possible
44 ///
45 /// Returns `None` if the streams in the given direction are currently exhausted.
46 pub fn open(&mut self, dir: Dir) -> Option<StreamId> {
47 if self.conn_state.is_closed() {
48 return None;
49 }
50
51 // TODO: Queue STREAM_ID_BLOCKED if this fails
52 if self.state.next[dir as usize] >= self.state.max[dir as usize] {
53 return None;
54 }
55
56 self.state.next[dir as usize] += 1;
57 let id = StreamId::new(self.state.side, dir, self.state.next[dir as usize] - 1);
58 self.state.insert(false, id);
59 self.state.send_streams += 1;
60 Some(id)
61 }
62
63 /// Accept a remotely initiated stream of a certain directionality, if possible
64 ///
65 /// Returns `None` if there are no new incoming streams for this connection.
66 /// Has no impact on the data flow-control or stream concurrency limits.
67 pub fn accept(&mut self, dir: Dir) -> Option<StreamId> {
68 if self.state.next_remote[dir as usize] == self.state.next_reported_remote[dir as usize] {
69 return None;
70 }
71
72 let x = self.state.next_reported_remote[dir as usize];
73 self.state.next_reported_remote[dir as usize] = x + 1;
74 if dir == Dir::Bi {
75 self.state.send_streams += 1;
76 }
77
78 Some(StreamId::new(!self.state.side, dir, x))
79 }
80
81 #[cfg(fuzzing)]
82 pub fn state(&mut self) -> &mut StreamsState {
83 self.state
84 }
85
86 /// The number of streams that may have unacknowledged data.
87 pub fn send_streams(&self) -> usize {
88 self.state.send_streams
89 }
90
91 /// The number of remotely initiated open streams of a certain directionality.
92 ///
93 /// Includes remotely initiated streams, which have not been accepted via [`accept`](Self::accept).
94 /// These streams count against the respective concurrency limit reported by
95 /// [`Connection::max_concurrent_streams`](super::Connection::max_concurrent_streams).
96 pub fn remote_open_streams(&self, dir: Dir) -> u64 {
97 // total opened - total closed = total opened - ( total permitted - total permitted unclosed )
98 self.state.next_remote[dir as usize]
99 - (self.state.max_remote[dir as usize]
100 - self.state.allocated_remote_count[dir as usize])
101 }
102}
103
104/// Access to streams
105pub struct RecvStream<'a> {
106 pub(super) id: StreamId,
107 pub(super) state: &'a mut StreamsState,
108 pub(super) pending: &'a mut Retransmits,
109}
110
111impl RecvStream<'_> {
112 /// Read from the given recv stream
113 ///
114 /// `max_length` limits the maximum size of the returned `Bytes` value; passing `usize::MAX`
115 /// will yield the best performance. `ordered` will make sure the returned chunk's offset will
116 /// have an offset exactly equal to the previously returned offset plus the previously returned
117 /// bytes' length.
118 ///
119 /// Yields `Ok(None)` if the stream was finished. Otherwise, yields a segment of data and its
120 /// offset in the stream. If `ordered` is `false`, segments may be received in any order, and
121 /// the `Chunk`'s `offset` field can be used to determine ordering in the caller.
122 ///
123 /// While most applications will prefer to consume stream data in order, unordered reads can
124 /// improve performance when packet loss occurs and data cannot be retransmitted before the flow
125 /// control window is filled. On any given stream, you can switch from ordered to unordered
126 /// reads, but ordered reads on streams that have seen previous unordered reads will return
127 /// `ReadError::IllegalOrderedRead`.
128 pub fn read(&mut self, ordered: bool) -> Result<Chunks, ReadableError> {
129 Chunks::new(self.id, ordered, self.state, self.pending)
130 }
131
132 /// Stop accepting data on the given receive stream
133 ///
134 /// Discards unread data and notifies the peer to stop transmitting. Once stopped, further
135 /// attempts to operate on a stream will yield `ClosedStream` errors.
136 pub fn stop(&mut self, error_code: VarInt) -> Result<(), ClosedStream> {
137 let mut entry = match self.state.recv.entry(self.id) {
138 hash_map::Entry::Occupied(s) => s,
139 hash_map::Entry::Vacant(_) => return Err(ClosedStream { _private: () }),
140 };
141 let stream = get_or_insert_recv(self.state.stream_receive_window)(entry.get_mut());
142
143 let (read_credits, stop_sending) = stream.stop()?;
144 if stop_sending.should_transmit() {
145 self.pending.stop_sending.push(frame::StopSending {
146 id: self.id,
147 error_code,
148 });
149 }
150
151 // We need to keep stopped streams around until they're finished or reset so we can update
152 // connection-level flow control to account for discarded data. Otherwise, we can discard
153 // state immediately.
154 if !stream.final_offset_unknown() {
155 let recv = entry.remove().expect("must have recv when stopping");
156 self.state.stream_recv_freed(self.id, recv);
157 }
158
159 if self.state.add_read_credits(read_credits).should_transmit() {
160 self.pending.max_data = true;
161 }
162
163 Ok(())
164 }
165
166 /// Check whether this stream has been reset by the peer, returning the reset error code if so
167 ///
168 /// After returning `Ok(Some(_))` once, stream state will be discarded and all future calls will
169 /// return `Err(ClosedStream)`.
170 pub fn received_reset(&mut self) -> Result<Option<VarInt>, ClosedStream> {
171 let hash_map::Entry::Occupied(entry) = self.state.recv.entry(self.id) else {
172 return Err(ClosedStream { _private: () });
173 };
174 let Some(s) = entry.get().as_ref().and_then(|s| s.as_open_recv()) else {
175 return Ok(None);
176 };
177 if s.stopped {
178 return Err(ClosedStream { _private: () });
179 }
180 let Some(code) = s.reset_code() else {
181 return Ok(None);
182 };
183
184 // Clean up state after application observes the reset, since there's no reason for the
185 // application to attempt to read or stop the stream once it knows it's reset
186 let (_, recv) = entry.remove_entry();
187 self.state
188 .stream_recv_freed(self.id, recv.expect("must have recv on reset"));
189 self.state.queue_max_stream_id(self.pending);
190
191 Ok(Some(code))
192 }
193}
194
195/// Access to streams
196pub struct SendStream<'a> {
197 pub(super) id: StreamId,
198 pub(super) state: &'a mut StreamsState,
199 pub(super) pending: &'a mut Retransmits,
200 pub(super) conn_state: &'a super::State,
201}
202
203#[allow(clippy::needless_lifetimes)] // Needed for cfg(fuzzing)
204impl<'a> SendStream<'a> {
205 #[cfg(fuzzing)]
206 pub fn new(
207 id: StreamId,
208 state: &'a mut StreamsState,
209 pending: &'a mut Retransmits,
210 conn_state: &'a super::State,
211 ) -> Self {
212 Self {
213 id,
214 state,
215 pending,
216 conn_state,
217 }
218 }
219
220 /// Send data on the given stream
221 ///
222 /// Returns the number of bytes successfully written.
223 pub fn write(&mut self, data: &[u8]) -> Result<usize, WriteError> {
224 Ok(self.write_source(&mut ByteSlice::from_slice(data))?.bytes)
225 }
226
227 /// Send data on the given stream
228 ///
229 /// Returns the number of bytes and chunks successfully written.
230 /// Note that this method might also write a partial chunk. In this case
231 /// [`Written::chunks`] will not count this chunk as fully written. However
232 /// the chunk will be advanced and contain only non-written data after the call.
233 pub fn write_chunks(&mut self, data: &mut [Bytes]) -> Result<Written, WriteError> {
234 self.write_source(&mut BytesArray::from_chunks(data))
235 }
236
237 fn write_source<B: BytesSource>(&mut self, source: &mut B) -> Result<Written, WriteError> {
238 if self.conn_state.is_closed() {
239 trace!(%self.id, "write blocked; connection draining");
240 return Err(WriteError::Blocked);
241 }
242
243 let limit = self.state.write_limit();
244
245 let max_send_data = self.state.max_send_data(self.id);
246
247 let stream = self
248 .state
249 .send
250 .get_mut(&self.id)
251 .map(get_or_insert_send(max_send_data))
252 .ok_or(WriteError::ClosedStream)?;
253
254 if limit == 0 {
255 trace!(
256 stream = %self.id, max_data = self.state.max_data, data_sent = self.state.data_sent,
257 "write blocked by connection-level flow control or send window"
258 );
259 if !stream.connection_blocked {
260 stream.connection_blocked = true;
261 self.state.connection_blocked.push(self.id);
262 }
263 return Err(WriteError::Blocked);
264 }
265
266 let was_pending = stream.is_pending();
267 let written = stream.write(source, limit)?;
268 self.state.data_sent += written.bytes as u64;
269 self.state.unacked_data += written.bytes as u64;
270 trace!(stream = %self.id, "wrote {} bytes", written.bytes);
271 if !was_pending {
272 self.state.pending.push_pending(self.id, stream.priority);
273 }
274 Ok(written)
275 }
276
277 /// Check if this stream was stopped, get the reason if it was
278 pub fn stopped(&self) -> Result<Option<VarInt>, ClosedStream> {
279 match self.state.send.get(&self.id).as_ref() {
280 Some(Some(s)) => Ok(s.stop_reason),
281 Some(None) => Ok(None),
282 None => Err(ClosedStream { _private: () }),
283 }
284 }
285
286 /// Finish a send stream, signalling that no more data will be sent.
287 ///
288 /// If this fails, no [`StreamEvent::Finished`] will be generated.
289 ///
290 /// [`StreamEvent::Finished`]: crate::StreamEvent::Finished
291 pub fn finish(&mut self) -> Result<(), FinishError> {
292 let max_send_data = self.state.max_send_data(self.id);
293 let stream = self
294 .state
295 .send
296 .get_mut(&self.id)
297 .map(get_or_insert_send(max_send_data))
298 .ok_or(FinishError::ClosedStream)?;
299
300 let was_pending = stream.is_pending();
301 stream.finish()?;
302 if !was_pending {
303 self.state.pending.push_pending(self.id, stream.priority);
304 }
305
306 Ok(())
307 }
308
309 /// Abandon transmitting data on a stream
310 ///
311 /// # Panics
312 /// - when applied to a receive stream
313 pub fn reset(&mut self, error_code: VarInt) -> Result<(), ClosedStream> {
314 let max_send_data = self.state.max_send_data(self.id);
315 let stream = self
316 .state
317 .send
318 .get_mut(&self.id)
319 .map(get_or_insert_send(max_send_data))
320 .ok_or(ClosedStream { _private: () })?;
321
322 if matches!(stream.state, SendState::ResetSent) {
323 // Redundant reset call
324 return Err(ClosedStream { _private: () });
325 }
326
327 // Restore the portion of the send window consumed by the data that we aren't about to
328 // send. We leave flow control alone because the peer's responsible for issuing additional
329 // credit based on the final offset communicated in the RESET_STREAM frame we send.
330 self.state.unacked_data -= stream.pending.unacked();
331 stream.reset();
332 self.pending.reset_stream.push((self.id, error_code));
333
334 // Don't reopen an already-closed stream we haven't forgotten yet
335 Ok(())
336 }
337
338 /// Set the priority of a stream
339 ///
340 /// # Panics
341 /// - when applied to a receive stream
342 pub fn set_priority(&mut self, priority: i32) -> Result<(), ClosedStream> {
343 let max_send_data = self.state.max_send_data(self.id);
344 let stream = self
345 .state
346 .send
347 .get_mut(&self.id)
348 .map(get_or_insert_send(max_send_data))
349 .ok_or(ClosedStream { _private: () })?;
350
351 stream.priority = priority;
352 Ok(())
353 }
354
355 /// Get the priority of a stream
356 ///
357 /// # Panics
358 /// - when applied to a receive stream
359 pub fn priority(&self) -> Result<i32, ClosedStream> {
360 let stream = self
361 .state
362 .send
363 .get(&self.id)
364 .ok_or(ClosedStream { _private: () })?;
365
366 Ok(stream.as_ref().map(|s| s.priority).unwrap_or_default())
367 }
368}
369
370/// A queue of streams with pending outgoing data, sorted by priority
371struct PendingStreamsQueue {
372 streams: BinaryHeap<PendingStream>,
373 /// The next stream to write out. This is `Some` when `TransportConfig::send_fairness(false)` and writing a stream is
374 /// interrupted while the stream still has some pending data. See `reinsert_pending()`.
375 next: Option<PendingStream>,
376 /// A monotonically decreasing counter, used to implement round-robin scheduling for streams of the same priority.
377 /// Underflowing is not a practical concern, as it is initialized to u64::MAX and only decremented by 1 in `push_pending`
378 recency: u64,
379}
380
381impl PendingStreamsQueue {
382 fn new() -> Self {
383 Self {
384 streams: BinaryHeap::new(),
385 next: None,
386 recency: u64::MAX,
387 }
388 }
389
390 /// Reinsert a stream that was pending and still contains unsent data.
391 fn reinsert_pending(&mut self, id: StreamId, priority: i32) {
392 assert!(self.next.is_none());
393
394 self.next = Some(PendingStream {
395 priority,
396 recency: self.recency, // the value here doesn't really matter
397 id,
398 });
399 }
400
401 /// Push a pending stream ID with the given priority, queued after any already-queued streams for the priority
402 fn push_pending(&mut self, id: StreamId, priority: i32) {
403 // Note that in the case where fairness is disabled, if we have a reinserted stream we don't
404 // bump it even if priority > next.priority. In order to minimize fragmentation we
405 // always try to complete a stream once part of it has been written.
406
407 // As the recency counter is monotonically decreasing, we know that using its value to sort this stream will queue it
408 // after all other queued streams of the same priority.
409 // This is enough to implement round-robin scheduling for streams that are still pending even after being handled,
410 // as in that case they are removed from the `BinaryHeap`, handled, and then immediately reinserted.
411 self.recency -= 1;
412 self.streams.push(PendingStream {
413 priority,
414 recency: self.recency,
415 id,
416 });
417 }
418
419 fn pop(&mut self) -> Option<PendingStream> {
420 self.next.take().or_else(|| self.streams.pop())
421 }
422
423 fn clear(&mut self) {
424 self.next = None;
425 self.streams.clear();
426 }
427
428 fn iter(&self) -> impl Iterator<Item = &PendingStream> {
429 self.next.iter().chain(self.streams.iter())
430 }
431
432 #[cfg(test)]
433 fn len(&self) -> usize {
434 self.streams.len() + self.next.is_some() as usize
435 }
436}
437
438/// The [`StreamId`] of a stream with pending data queued, ordered by its priority and recency
439#[derive(Clone, PartialEq, Eq, PartialOrd, Ord)]
440struct PendingStream {
441 /// The priority of the stream
442 // Note that this field should be kept above the `recency` field, in order for the `Ord` derive to be correct
443 // (See https://doc.rust-lang.org/stable/std/cmp/trait.Ord.html#derivable)
444 priority: i32,
445 /// A tie-breaker for streams of the same priority, used to improve fairness by implementing round-robin scheduling:
446 /// Larger values are prioritized, so it is initialised to `u64::MAX`, and when a stream writes data, we know
447 /// that it currently has the highest recency value, so it is deprioritized by setting its recency to 1 less than the
448 /// previous lowest recency value, such that all other streams of this priority will get processed once before we get back
449 /// round to this one
450 recency: u64,
451 /// The ID of the stream
452 // The way this type is used ensures that every instance has a unique `recency` value, so this field should be kept below
453 // the `priority` and `recency` fields, so that it does not interfere with the behaviour of the `Ord` derive
454 id: StreamId,
455}
456
457/// Application events about streams
458#[derive(Debug, PartialEq, Eq)]
459pub enum StreamEvent {
460 /// One or more new streams has been opened and might be readable
461 Opened {
462 /// Directionality for which streams have been opened
463 dir: Dir,
464 },
465 /// A currently open stream likely has data or errors waiting to be read
466 Readable {
467 /// Which stream is now readable
468 id: StreamId,
469 },
470 /// A formerly write-blocked stream might be ready for a write or have been stopped
471 ///
472 /// Only generated for streams that are currently open.
473 Writable {
474 /// Which stream is now writable
475 id: StreamId,
476 },
477 /// A finished stream has been fully acknowledged or stopped
478 Finished {
479 /// Which stream has been finished
480 id: StreamId,
481 },
482 /// The peer asked us to stop sending on an outgoing stream
483 Stopped {
484 /// Which stream has been stopped
485 id: StreamId,
486 /// Error code supplied by the peer
487 error_code: VarInt,
488 },
489 /// At least one new stream of a certain directionality may be opened
490 Available {
491 /// Directionality for which streams are newly available
492 dir: Dir,
493 },
494}
495
496/// Indicates whether a frame needs to be transmitted
497///
498/// This type wraps around bool and uses the `#[must_use]` attribute in order
499/// to prevent accidental loss of the frame transmission requirement.
500#[derive(Copy, Clone, Debug, Default, Eq, PartialEq)]
501#[must_use = "A frame might need to be enqueued"]
502pub struct ShouldTransmit(bool);
503
504impl ShouldTransmit {
505 /// Returns whether a frame should be transmitted
506 pub fn should_transmit(self) -> bool {
507 self.0
508 }
509}
510
511/// Error indicating that a stream has not been opened or has already been finished or reset
512#[derive(Debug, Default, Error, Clone, PartialEq, Eq)]
513#[error("closed stream")]
514pub struct ClosedStream {
515 _private: (),
516}
517
518impl From<ClosedStream> for io::Error {
519 fn from(x: ClosedStream) -> Self {
520 Self::new(io::ErrorKind::NotConnected, x)
521 }
522}
523
524#[derive(Debug, Copy, Clone, Eq, PartialEq)]
525enum StreamHalf {
526 Send,
527 Recv,
528}