This is an automated email from the ASF dual-hosted git repository.
alamb pushed a commit to branch main
in repository https://gitbox.apache.org/repos/asf/datafusion.git
The following commit(s) were added to refs/heads/main by this push:
new 532bd14992 doc: update
RecordBatchReceiverStreamBuilder::spawn_blocking task behaviour (#14995)
532bd14992 is described below
commit 532bd14992a87f841b4b9dbafa278305fef4c964
Author: Shruti Sharma <[email protected]>
AuthorDate: Sat Mar 8 14:59:46 2025 +0530
doc: update RecordBatchReceiverStreamBuilder::spawn_blocking task behaviour
(#14995)
* update doc for RecordBatchReceiverStreamBuilder::spawn_blocking
* Clarify comments
---------
Co-authored-by: Andrew Lamb <[email protected]>
---
datafusion/physical-plan/src/stream.rs | 21 +++++++++++++++++++--
1 file changed, 19 insertions(+), 2 deletions(-)
diff --git a/datafusion/physical-plan/src/stream.rs
b/datafusion/physical-plan/src/stream.rs
index 23cbb1ce49..f2324dfd9b 100644
--- a/datafusion/physical-plan/src/stream.rs
+++ b/datafusion/physical-plan/src/stream.rs
@@ -223,6 +223,10 @@ impl RecordBatchReceiverStreamBuilder {
}
/// Get a handle for sending [`RecordBatch`] to the output
+ ///
+ /// If the stream is dropped / canceled, the sender will be closed and
+ /// calling `tx().send()` will return an error. Producers should stop
+ /// producing in this case and return control.
pub fn tx(&self) -> Sender<Result<RecordBatch>> {
self.inner.tx()
}
@@ -241,8 +245,21 @@ impl RecordBatchReceiverStreamBuilder {
self.inner.spawn(task)
}
- /// Spawn a blocking task that will be aborted if this builder (or the
stream
- /// built from it) are dropped
+ /// Spawn a blocking task tied to the builder and stream.
+ ///
+ /// # Drop / Cancel Behavior
+ ///
+ /// If this builder (or the stream built from it) is dropped **before** the
+ /// task starts, the task is also dropped and will never start execute.
+ ///
+ /// **Note:** Once the blocking task has started, it **will not** be
+ /// forcibly stopped on drop as Rust does not allow forcing a running
thread
+ /// to terminate. The task will continue running until it completes or
+ /// encounters an error.
+ ///
+ /// Users should ensure that their blocking function periodically checks
for
+ /// errors calling `tx.blocking_send`. An error signals that the stream has
+ /// been dropped / cancelled and the blocking task should exit.
///
/// This is often used to spawn tasks that write to the sender
/// retrieved from [`Self::tx`], for examples, see the document
---------------------------------------------------------------------
To unsubscribe, e-mail: [email protected]
For additional commands, e-mail: [email protected]