Skip to content

Commit

Permalink
Support LogicalPlan Debug differently than Display (#11774)
Browse files Browse the repository at this point in the history
* Derive Debug for logical plan nodes

* Improve LogicalPlan debug printing

* Fix tests

* Fix tests

* Fix tests
  • Loading branch information
lewiszlw authored Aug 4, 2024
1 parent f4e519f commit a4d41d6
Show file tree
Hide file tree
Showing 30 changed files with 211 additions and 186 deletions.
4 changes: 2 additions & 2 deletions benchmarks/src/tpch/run.rs
Original file line number Diff line number Diff line change
Expand Up @@ -205,12 +205,12 @@ impl RunOpt {
let (state, plan) = plan.into_parts();

if debug {
println!("=== Logical plan ===\n{plan:?}\n");
println!("=== Logical plan ===\n{plan}\n");
}

let plan = state.optimize(&plan)?;
if debug {
println!("=== Optimized logical plan ===\n{plan:?}\n");
println!("=== Optimized logical plan ===\n{plan}\n");
}
let physical_plan = state.create_physical_plan(&plan).await?;
if debug {
Expand Down
14 changes: 7 additions & 7 deletions datafusion/core/src/dataframe/mod.rs
Original file line number Diff line number Diff line change
Expand Up @@ -2553,7 +2553,7 @@ mod tests {
\n TableScan: a\
\n Projection: b.c1, b.c2\
\n TableScan: b";
assert_eq!(expected_plan, format!("{:?}", join.logical_plan()));
assert_eq!(expected_plan, format!("{}", join.logical_plan()));

Ok(())
}
Expand All @@ -2572,7 +2572,7 @@ mod tests {
let expected_plan = "CrossJoin:\
\n TableScan: a projection=[c1], full_filters=[Boolean(NULL)]\
\n TableScan: b projection=[c1]";
assert_eq!(expected_plan, format!("{:?}", join.into_optimized_plan()?));
assert_eq!(expected_plan, format!("{}", join.into_optimized_plan()?));

// JOIN ON expression must be boolean type
let join = left.join_on(right, JoinType::Inner, Some(lit("TRUE")))?;
Expand Down Expand Up @@ -2914,7 +2914,7 @@ mod tests {
\n Inner Join: t1.c1 = t2.c1\
\n TableScan: t1\
\n TableScan: t2",
format!("{:?}", df_with_column.logical_plan())
format!("{}", df_with_column.logical_plan())
);

assert_eq!(
Expand All @@ -2927,7 +2927,7 @@ mod tests {
\n TableScan: aggregate_test_100 projection=[c1]\
\n SubqueryAlias: t2\
\n TableScan: aggregate_test_100 projection=[c1]",
format!("{:?}", df_with_column.clone().into_optimized_plan()?)
format!("{}", df_with_column.clone().into_optimized_plan()?)
);

let df_results = df_with_column.collect().await?;
Expand Down Expand Up @@ -3109,7 +3109,7 @@ mod tests {
\n Inner Join: t1.c1 = t2.c1\
\n TableScan: t1\
\n TableScan: t2",
format!("{:?}", df_renamed.logical_plan())
format!("{}", df_renamed.logical_plan())
);

assert_eq!("\
Expand All @@ -3121,7 +3121,7 @@ mod tests {
\n TableScan: aggregate_test_100 projection=[c1, c2, c3]\
\n SubqueryAlias: t2\
\n TableScan: aggregate_test_100 projection=[c1, c2, c3]",
format!("{:?}", df_renamed.clone().into_optimized_plan()?)
format!("{}", df_renamed.clone().into_optimized_plan()?)
);

let df_results = df_renamed.collect().await?;
Expand Down Expand Up @@ -3306,7 +3306,7 @@ mod tests {

assert_eq!(
"TableScan: ?table? projection=[c2, c3, sum]",
format!("{:?}", cached_df.clone().into_optimized_plan()?)
format!("{}", cached_df.clone().into_optimized_plan()?)
);

let df_results = df.collect().await?;
Expand Down
2 changes: 1 addition & 1 deletion datafusion/core/tests/custom_sources_cases/mod.rs
Original file line number Diff line number Diff line change
Expand Up @@ -246,7 +246,7 @@ async fn custom_source_dataframe() -> Result<()> {
}

let expected = format!("TableScan: {UNNAMED_TABLE} projection=[c2]");
assert_eq!(format!("{optimized_plan:?}"), expected);
assert_eq!(format!("{optimized_plan}"), expected);

let physical_plan = state.create_physical_plan(&optimized_plan).await?;

Expand Down
2 changes: 1 addition & 1 deletion datafusion/core/tests/expr_api/simplification.rs
Original file line number Diff line number Diff line change
Expand Up @@ -119,7 +119,7 @@ fn get_optimized_plan_formatted(plan: LogicalPlan, date_time: &DateTime<Utc>) ->
let optimizer = Optimizer::with_rules(vec![Arc::new(SimplifyExpressions::new())]);
let optimized_plan = optimizer.optimize(plan, &config, observe).unwrap();

format!("{optimized_plan:?}")
format!("{optimized_plan}")
}

// ------------------------------
Expand Down
5 changes: 2 additions & 3 deletions datafusion/core/tests/optimizer_integration.rs
Original file line number Diff line number Diff line change
Expand Up @@ -81,14 +81,13 @@ fn timestamp_nano_ts_utc_predicates() {
let sql = "SELECT col_int32
FROM test
WHERE col_ts_nano_utc < (now() - interval '1 hour')";
let plan = test_sql(sql).unwrap();
// a scan should have the now()... predicate folded to a single
// constant and compared to the column without a cast so it can be
// pushed down / pruned
let expected =
"Projection: test.col_int32\n Filter: test.col_ts_nano_utc < TimestampNanosecond(1666612093000000000, Some(\"+00:00\"))\
\n TableScan: test projection=[col_int32, col_ts_nano_utc]";
assert_eq!(expected, format!("{plan:?}"));
quick_test(sql, expected);
}

#[test]
Expand Down Expand Up @@ -117,7 +116,7 @@ fn concat_ws_literals() -> Result<()> {

fn quick_test(sql: &str, expected_plan: &str) {
let plan = test_sql(sql).unwrap();
assert_eq!(expected_plan, format!("{:?}", plan));
assert_eq!(expected_plan, format!("{}", plan));
}

fn test_sql(sql: &str) -> Result<LogicalPlan> {
Expand Down
6 changes: 3 additions & 3 deletions datafusion/core/tests/sql/explain_analyze.rs
Original file line number Diff line number Diff line change
Expand Up @@ -253,7 +253,7 @@ async fn csv_explain_plans() {

// Optimized logical plan
let state = ctx.state();
let msg = format!("Optimizing logical plan for '{sql}': {plan:?}");
let msg = format!("Optimizing logical plan for '{sql}': {plan}");
let plan = state.optimize(plan).expect(&msg);
let optimized_logical_schema = plan.schema();
// Both schema has to be the same
Expand Down Expand Up @@ -327,7 +327,7 @@ async fn csv_explain_plans() {

// Physical plan
// Create plan
let msg = format!("Creating physical plan for '{sql}': {plan:?}");
let msg = format!("Creating physical plan for '{sql}': {plan}");
let plan = state.create_physical_plan(&plan).await.expect(&msg);
//
// Execute plan
Expand Down Expand Up @@ -548,7 +548,7 @@ async fn csv_explain_verbose_plans() {

// Physical plan
// Create plan
let msg = format!("Creating physical plan for '{sql}': {plan:?}");
let msg = format!("Creating physical plan for '{sql}': {plan}");
let plan = state.create_physical_plan(&plan).await.expect(&msg);
//
// Execute plan
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -413,7 +413,7 @@ async fn test_parameterized_aggregate_udf() -> Result<()> {
.build()?;

assert_eq!(
format!("{plan:?}"),
format!("{plan}"),
"Aggregate: groupBy=[[t.text]], aggr=[[geo_mean(t.text) AS a, geo_mean(t.text) AS b]]\n TableScan: t projection=[text]"
);

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -139,7 +139,7 @@ async fn scalar_udf() -> Result<()> {
.build()?;

assert_eq!(
format!("{plan:?}"),
format!("{plan}"),
"Projection: t.a, t.b, my_add(t.a, t.b)\n TableScan: t projection=[a, b]"
);

Expand Down Expand Up @@ -393,7 +393,7 @@ async fn udaf_as_window_func() -> Result<()> {
TableScan: my_table"#;

let dataframe = context.sql(sql).await.unwrap();
assert_eq!(format!("{:?}", dataframe.logical_plan()), expected);
assert_eq!(format!("{}", dataframe.logical_plan()), expected);
Ok(())
}

Expand Down Expand Up @@ -1124,7 +1124,7 @@ async fn test_parameterized_scalar_udf() -> Result<()> {
.build()?;

assert_eq!(
format!("{plan:?}"),
format!("{plan}"),
"Filter: t.text IS NOT NULL\n Filter: regex_udf(t.text) AND regex_udf(t.text)\n TableScan: t projection=[text]"
);

Expand Down
30 changes: 15 additions & 15 deletions datafusion/expr/src/logical_plan/builder.rs
Original file line number Diff line number Diff line change
Expand Up @@ -1749,7 +1749,7 @@ mod tests {
\n Filter: employee_csv.state = Utf8(\"CO\")\
\n TableScan: employee_csv projection=[id, state]";

assert_eq!(expected, format!("{plan:?}"));
assert_eq!(expected, format!("{plan}"));

Ok(())
}
Expand Down Expand Up @@ -1802,7 +1802,7 @@ mod tests {
let expected = "Sort: employee_csv.state ASC NULLS FIRST, employee_csv.salary DESC NULLS LAST\
\n TableScan: employee_csv projection=[state, salary]";

assert_eq!(expected, format!("{plan:?}"));
assert_eq!(expected, format!("{plan}"));

Ok(())
}
Expand All @@ -1822,7 +1822,7 @@ mod tests {
\n TableScan: t1\
\n TableScan: t2";

assert_eq!(expected, format!("{plan:?}"));
assert_eq!(expected, format!("{plan}"));

Ok(())
}
Expand All @@ -1847,7 +1847,7 @@ mod tests {
\n TableScan: employee_csv projection=[state, salary]\
\n TableScan: employee_csv projection=[state, salary]";

assert_eq!(expected, format!("{plan:?}"));
assert_eq!(expected, format!("{plan}"));

Ok(())
}
Expand Down Expand Up @@ -1876,7 +1876,7 @@ mod tests {
\n TableScan: employee_csv projection=[state, salary]\
\n TableScan: employee_csv projection=[state, salary]";

assert_eq!(expected, format!("{plan:?}"));
assert_eq!(expected, format!("{plan}"));

Ok(())
}
Expand Down Expand Up @@ -1913,7 +1913,7 @@ mod tests {
\n Filter: employee_csv.state = Utf8(\"CO\")\
\n TableScan: employee_csv projection=[id, state]";

assert_eq!(expected, format!("{plan:?}"));
assert_eq!(expected, format!("{plan}"));

Ok(())
}
Expand All @@ -1940,7 +1940,7 @@ mod tests {
\n TableScan: foo\
\n Projection: bar.a\
\n TableScan: bar";
assert_eq!(expected, format!("{outer_query:?}"));
assert_eq!(expected, format!("{outer_query}"));

Ok(())
}
Expand Down Expand Up @@ -1968,7 +1968,7 @@ mod tests {
\n TableScan: foo\
\n Projection: bar.a\
\n TableScan: bar";
assert_eq!(expected, format!("{outer_query:?}"));
assert_eq!(expected, format!("{outer_query}"));

Ok(())
}
Expand All @@ -1994,7 +1994,7 @@ mod tests {
\n Projection: foo.b\
\n TableScan: foo\
\n TableScan: bar";
assert_eq!(expected, format!("{outer_query:?}"));
assert_eq!(expected, format!("{outer_query}"));

Ok(())
}
Expand Down Expand Up @@ -2116,7 +2116,7 @@ mod tests {
let expected = "\
Unnest: lists[test_table.strings] structs[]\
\n TableScan: test_table";
assert_eq!(expected, format!("{plan:?}"));
assert_eq!(expected, format!("{plan}"));

// Check unnested field is a scalar
let field = plan.schema().field_with_name(None, "strings").unwrap();
Expand All @@ -2130,7 +2130,7 @@ mod tests {
let expected = "\
Unnest: lists[] structs[test_table.struct_singular]\
\n TableScan: test_table";
assert_eq!(expected, format!("{plan:?}"));
assert_eq!(expected, format!("{plan}"));

for field_name in &["a", "b"] {
// Check unnested struct field is a scalar
Expand All @@ -2153,7 +2153,7 @@ mod tests {
\n Unnest: lists[test_table.structs] structs[]\
\n Unnest: lists[test_table.strings] structs[]\
\n TableScan: test_table";
assert_eq!(expected, format!("{plan:?}"));
assert_eq!(expected, format!("{plan}"));

// Check unnested struct list field should be a struct.
let field = plan.schema().field_with_name(None, "structs").unwrap();
Expand All @@ -2171,7 +2171,7 @@ mod tests {
let expected = "\
Unnest: lists[test_table.strings, test_table.structs] structs[test_table.struct_singular]\
\n TableScan: test_table";
assert_eq!(expected, format!("{plan:?}"));
assert_eq!(expected, format!("{plan}"));

// Unnesting missing column should fail.
let plan = nested_table_scan("test_table")?.unnest_column("missing");
Expand Down Expand Up @@ -2263,9 +2263,9 @@ mod tests {
])?
.build()?;

let plan_expected = format!("{plan:?}");
let plan_expected = format!("{plan}");
let plan_builder: LogicalPlanBuilder = Arc::new(plan).into();
assert_eq!(plan_expected, format!("{:?}", plan_builder.plan));
assert_eq!(plan_expected, format!("{}", plan_builder.plan));

Ok(())
}
Expand Down
18 changes: 9 additions & 9 deletions datafusion/expr/src/logical_plan/ddl.rs
Original file line number Diff line number Diff line change
Expand Up @@ -29,7 +29,7 @@ use datafusion_common::{Constraints, DFSchemaRef, SchemaReference, TableReferenc
use sqlparser::ast::Ident;

/// Various types of DDL (CREATE / DROP) catalog manipulation
#[derive(Clone, PartialEq, Eq, Hash)]
#[derive(Debug, Clone, PartialEq, Eq, Hash)]
pub enum DdlStatement {
/// Creates an external table.
CreateExternalTable(CreateExternalTable),
Expand Down Expand Up @@ -179,7 +179,7 @@ impl DdlStatement {
}

/// Creates an external table.
#[derive(Clone, PartialEq, Eq)]
#[derive(Debug, Clone, PartialEq, Eq)]
pub struct CreateExternalTable {
/// The table schema
pub schema: DFSchemaRef,
Expand Down Expand Up @@ -224,7 +224,7 @@ impl Hash for CreateExternalTable {
}

/// Creates an in memory table.
#[derive(Clone, PartialEq, Eq, Hash)]
#[derive(Debug, Clone, PartialEq, Eq, Hash)]
pub struct CreateMemoryTable {
/// The table name
pub name: TableReference,
Expand All @@ -241,7 +241,7 @@ pub struct CreateMemoryTable {
}

/// Creates a view.
#[derive(Clone, PartialEq, Eq, Hash)]
#[derive(Debug, Clone, PartialEq, Eq, Hash)]
pub struct CreateView {
/// The table name
pub name: TableReference,
Expand All @@ -254,7 +254,7 @@ pub struct CreateView {
}

/// Creates a catalog (aka "Database").
#[derive(Clone, PartialEq, Eq, Hash)]
#[derive(Debug, Clone, PartialEq, Eq, Hash)]
pub struct CreateCatalog {
/// The catalog name
pub catalog_name: String,
Expand All @@ -265,7 +265,7 @@ pub struct CreateCatalog {
}

/// Creates a schema.
#[derive(Clone, PartialEq, Eq, Hash)]
#[derive(Debug, Clone, PartialEq, Eq, Hash)]
pub struct CreateCatalogSchema {
/// The table schema
pub schema_name: String,
Expand All @@ -276,7 +276,7 @@ pub struct CreateCatalogSchema {
}

/// Drops a table.
#[derive(Clone, PartialEq, Eq, Hash)]
#[derive(Debug, Clone, PartialEq, Eq, Hash)]
pub struct DropTable {
/// The table name
pub name: TableReference,
Expand All @@ -287,7 +287,7 @@ pub struct DropTable {
}

/// Drops a view.
#[derive(Clone, PartialEq, Eq, Hash)]
#[derive(Debug, Clone, PartialEq, Eq, Hash)]
pub struct DropView {
/// The view name
pub name: TableReference,
Expand All @@ -298,7 +298,7 @@ pub struct DropView {
}

/// Drops a schema
#[derive(Clone, PartialEq, Eq, Hash)]
#[derive(Debug, Clone, PartialEq, Eq, Hash)]
pub struct DropCatalogSchema {
/// The schema name
pub name: SchemaReference,
Expand Down
Loading

0 comments on commit a4d41d6

Please sign in to comment.