Skip to content

Commit 0109044

Browse files
committed
Add additional log on GHJ start
1 parent b9651d3 commit 0109044

1 file changed

Lines changed: 16 additions & 1 deletion

File tree

  • datafusion/physical-plan/src/joins/grace_hash_join

datafusion/physical-plan/src/joins/grace_hash_join/exec.rs

Lines changed: 16 additions & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -42,7 +42,7 @@ use std::any::Any;
4242
use std::fmt;
4343
use std::fmt::Formatter;
4444
use std::mem::size_of;
45-
use std::sync::{Arc, OnceLock};
45+
use std::sync::Arc;
4646

4747
use arrow::array::UInt32Array;
4848
use arrow::array::{Array, StringViewArray};
@@ -632,6 +632,21 @@ impl ExecutionPlan for GraceHashJoinExec {
632632

633633
let join_metrics = Arc::new(BuildProbeJoinMetrics::new(partition, &self.metrics));
634634

635+
// Debug: log pool usage at start of GHJ execute to diagnose out-of-memory conditions.
636+
let pool = context.memory_pool();
637+
let reserved_now = pool.reserved();
638+
let limit_msg = match pool.memory_limit() {
639+
MemoryLimit::Finite(l) => human_readable_size(l),
640+
MemoryLimit::Infinite => "inf".to_string(),
641+
MemoryLimit::Unknown => "unknown".to_string(),
642+
};
643+
debug!(
644+
"Grace hash join execute partition {} pool reserved={}, limit={}",
645+
partition,
646+
human_readable_size(reserved_now),
647+
limit_msg
648+
);
649+
635650
let left = self.left.execute(partition, Arc::clone(&context))?;
636651
let left_schema = Arc::clone(&self.left.schema());
637652
let on_left = self

0 commit comments

Comments
 (0)