Skip to content
Merged
Show file tree
Hide file tree
Changes from all commits
Commits
File filter

Filter by extension

Filter by extension

Conversations
Failed to load comments.
Loading
Jump to
Jump to file
Failed to load files.
Loading
Diff view
Diff view
28 changes: 14 additions & 14 deletions datafusion/core/src/optimizer/common_subexpr_eliminate.rs
Original file line number Diff line number Diff line change
Expand Up @@ -17,19 +17,18 @@

//! Eliminate common sub-expression.

use crate::error::Result;
use crate::logical_plan::plan::{Filter, Projection, Window};
use crate::logical_plan::{
use crate::optimizer::optimizer::{OptimizerConfig, OptimizerRule};
use arrow::datatypes::DataType;
use datafusion_common::{DFField, DFSchema, Result};
use datafusion_expr::{
col,
plan::{Aggregate, Sort},
DFField, DFSchema, Expr, ExprRewritable, ExprRewriter, ExprSchemable, ExprVisitable,
ExpressionVisitor, LogicalPlan, Recursion, RewriteRecursion,
expr::GroupingSet,
expr_rewriter::{ExprRewritable, ExprRewriter, RewriteRecursion},
expr_visitor::{ExprVisitable, ExpressionVisitor, Recursion},
logical_plan::{Aggregate, Filter, LogicalPlan, Projection, Sort, Window},
utils::from_plan,
Expr, ExprSchemable,
};
use crate::optimizer::optimizer::OptimizerConfig;
use crate::optimizer::optimizer::OptimizerRule;
use arrow::datatypes::DataType;
use datafusion_expr::expr::GroupingSet;
use datafusion_expr::utils::from_plan;
use std::collections::{HashMap, HashSet};
use std::sync::Arc;

Expand Down Expand Up @@ -696,10 +695,11 @@ fn replace_common_expr(
#[cfg(test)]
mod test {
use super::*;
use crate::logical_plan::{
avg, binary_expr, col, lit, sum, LogicalPlanBuilder, Operator,
};
use crate::test::*;
use datafusion_expr::{
avg, binary_expr, col, lit, logical_plan::builder::LogicalPlanBuilder, sum,
Operator,
};
use std::iter;

fn assert_optimized_plan_eq(plan: &LogicalPlan, expected: &str) {
Expand Down
19 changes: 8 additions & 11 deletions datafusion/core/src/optimizer/eliminate_filter.rs
Original file line number Diff line number Diff line change
Expand Up @@ -18,16 +18,14 @@
//! Optimizer rule to replace `where false` on a plan with an empty relation.
//! This saves time in planning and executing the query.
//! Note that this rule should be applied after simplify expressions optimizer rule.
use datafusion_common::ScalarValue;
use datafusion_expr::utils::from_plan;
use datafusion_expr::Expr;
use datafusion_common::{Result, ScalarValue};
use datafusion_expr::{
logical_plan::{EmptyRelation, Filter, LogicalPlan},
utils::from_plan,
Expr,
};

use crate::error::Result;
use crate::logical_plan::plan::Filter;
use crate::logical_plan::{EmptyRelation, LogicalPlan};
use crate::optimizer::optimizer::OptimizerRule;

use crate::optimizer::optimizer::OptimizerConfig;
use crate::optimizer::optimizer::{OptimizerConfig, OptimizerRule};

/// Optimization rule that elimanate the scalar value (true/false) filter with an [LogicalPlan::EmptyRelation]
#[derive(Default)]
Expand Down Expand Up @@ -81,9 +79,8 @@ impl OptimizerRule for EliminateFilter {
#[cfg(test)]
mod tests {
use super::*;
use crate::logical_plan::LogicalPlanBuilder;
use crate::logical_plan::{col, sum};
use crate::test::*;
use datafusion_expr::{col, logical_plan::builder::LogicalPlanBuilder, sum};

fn assert_optimized_plan_eq(plan: &LogicalPlan, expected: &str) {
let rule = EliminateFilter::new();
Expand Down
15 changes: 7 additions & 8 deletions datafusion/core/src/optimizer/eliminate_limit.rs
Original file line number Diff line number Diff line change
Expand Up @@ -17,12 +17,12 @@

//! Optimizer rule to replace `LIMIT 0` on a plan with an empty relation.
//! This saves time in planning and executing the query.
use crate::error::Result;
use crate::logical_plan::{EmptyRelation, Limit, LogicalPlan};
use crate::optimizer::optimizer::OptimizerRule;
use datafusion_expr::utils::from_plan;

use crate::optimizer::optimizer::OptimizerConfig;
use crate::optimizer::optimizer::{OptimizerConfig, OptimizerRule};
use datafusion_common::Result;
use datafusion_expr::{
logical_plan::{EmptyRelation, Limit, LogicalPlan},
utils::from_plan,
};

/// Optimization rule that replaces LIMIT 0 with an [LogicalPlan::EmptyRelation]
#[derive(Default)]
Expand Down Expand Up @@ -72,9 +72,8 @@ impl OptimizerRule for EliminateLimit {
#[cfg(test)]
mod tests {
use super::*;
use crate::logical_plan::LogicalPlanBuilder;
use crate::logical_plan::{col, sum};
use crate::test::*;
use datafusion_expr::{col, logical_plan::builder::LogicalPlanBuilder, sum};

fn assert_optimized_plan_eq(plan: &LogicalPlan, expected: &str) {
let rule = EliminateLimit::new();
Expand Down
14 changes: 7 additions & 7 deletions datafusion/core/src/optimizer/filter_push_down.rs
Original file line number Diff line number Diff line change
Expand Up @@ -14,8 +14,10 @@

//! Filter Push Down optimizer rule ensures that filters are applied as early as possible in the plan

use crate::optimizer::optimizer::OptimizerConfig;
use crate::optimizer::{optimizer::OptimizerRule, utils};
use crate::optimizer::{
optimizer::{OptimizerConfig, OptimizerRule},
utils,
};
use datafusion_common::{Column, DFSchema, Result};
use datafusion_expr::{
col,
Expand Down Expand Up @@ -560,19 +562,17 @@ fn rewrite(expr: &Expr, projection: &HashMap<String, Expr>) -> Result<Expr> {

#[cfg(test)]
mod tests {
use std::sync::Arc;

use super::*;
use crate::test::*;
use arrow::datatypes::SchemaRef;
use async_trait::async_trait;
use datafusion_common::DFSchema;
use datafusion_expr::{
and, col, lit,
logical_plan::{builder::union_with_alias, JoinType},
sum, Expr, LogicalPlanBuilder, Operator, TableSource, TableType,
};

use arrow::datatypes::SchemaRef;
use async_trait::async_trait;
use std::sync::Arc;

fn optimize_plan(plan: &LogicalPlan) -> LogicalPlan {
let rule = FilterPushDown::new();
Expand Down
27 changes: 13 additions & 14 deletions datafusion/core/src/optimizer/limit_push_down.rs
Original file line number Diff line number Diff line change
Expand Up @@ -17,15 +17,14 @@

//! Optimizer rule to push down LIMIT in the query plan
//! It will push down through projection, limits (taking the smaller limit)
use crate::error::Result;
use crate::logical_plan::plan::Projection;
use crate::logical_plan::{Limit, TableScan};
use crate::logical_plan::{LogicalPlan, Union};
use crate::optimizer::optimizer::OptimizerConfig;
use crate::optimizer::optimizer::OptimizerRule;
use datafusion_common::DataFusionError;
use datafusion_expr::logical_plan::{Join, JoinType, Offset};
use datafusion_expr::utils::from_plan;
use crate::optimizer::optimizer::{OptimizerConfig, OptimizerRule};
use datafusion_common::{DataFusionError, Result};
use datafusion_expr::{
logical_plan::{
Join, JoinType, Limit, LogicalPlan, Offset, Projection, TableScan, Union,
},
utils::from_plan,
};
use std::sync::Arc;

/// Optimization rule that tries pushes down LIMIT n
Expand Down Expand Up @@ -270,12 +269,12 @@ impl OptimizerRule for LimitPushDown {
#[cfg(test)]
mod test {
use super::*;
use crate::{
logical_plan::{col, max, LogicalPlan, LogicalPlanBuilder},
test::*,
use crate::test::*;
use datafusion_expr::{
col, exists,
logical_plan::{builder::LogicalPlanBuilder, JoinType, LogicalPlan},
max,
};
use datafusion_expr::exists;
use datafusion_expr::logical_plan::JoinType;

fn assert_optimized_plan_eq(plan: &LogicalPlan, expected: &str) {
let rule = LimitPushDown::new();
Expand Down
8 changes: 3 additions & 5 deletions datafusion/core/src/optimizer/optimizer.rs
Original file line number Diff line number Diff line change
Expand Up @@ -18,12 +18,10 @@
//! Query optimizer traits

use chrono::{DateTime, Utc};
use std::sync::Arc;

use datafusion_common::Result;
use datafusion_expr::logical_plan::LogicalPlan;
use log::{debug, trace};

use crate::error::Result;
use crate::logical_plan::LogicalPlan;
use std::sync::Arc;

/// `OptimizerRule` transforms one ['LogicalPlan'] into another which
/// computes the same results, but in a potentially more efficient
Expand Down
37 changes: 20 additions & 17 deletions datafusion/core/src/optimizer/projection_push_down.rs
Original file line number Diff line number Diff line change
Expand Up @@ -18,22 +18,21 @@
//! Projection Push Down optimizer rule ensures that only referenced columns are
//! loaded into memory

use crate::error::{DataFusionError, Result};
use crate::logical_plan::plan::{
Aggregate, Analyze, Join, Projection, SubqueryAlias, TableScan, Window,
};
use crate::logical_plan::{
build_join_schema, Column, DFField, DFSchema, DFSchemaRef, LogicalPlan,
LogicalPlanBuilder, ToDFSchema, Union,
};
use crate::optimizer::optimizer::OptimizerConfig;
use crate::optimizer::optimizer::OptimizerRule;
use crate::optimizer::optimizer::{OptimizerConfig, OptimizerRule};
use arrow::datatypes::{Field, Schema};
use arrow::error::Result as ArrowResult;
use datafusion_expr::utils::{
expr_to_columns, exprlist_to_columns, find_sort_exprs, from_plan,
use datafusion_common::{
Column, DFField, DFSchema, DFSchemaRef, DataFusionError, Result, ToDFSchema,
};
use datafusion_expr::{
logical_plan::{
builder::{build_join_schema, LogicalPlanBuilder},
Aggregate, Analyze, Join, LogicalPlan, Projection, SubqueryAlias, TableScan,
Union, Window,
},
utils::{expr_to_columns, exprlist_to_columns, find_sort_exprs, from_plan},
Expr,
};
use datafusion_expr::Expr;
use std::{
collections::{BTreeSet, HashSet},
sync::Arc,
Expand Down Expand Up @@ -529,14 +528,18 @@ fn optimize_plan(
#[cfg(test)]
mod tests {

use std::collections::HashMap;

use super::*;
use crate::logical_plan::{col, lit, max, min, Expr, JoinType, LogicalPlanBuilder};
use crate::test::*;
use crate::test_util::scan_empty;
use arrow::datatypes::DataType;
use datafusion_expr::utils::exprlist_to_fields;
use datafusion_expr::{
col, lit,
logical_plan::{builder::LogicalPlanBuilder, JoinType},
max, min,
utils::exprlist_to_fields,
Expr,
};
use std::collections::HashMap;

#[test]
fn aggregate_no_group_by() -> Result<()> {
Expand Down
45 changes: 22 additions & 23 deletions datafusion/core/src/optimizer/simplify_expressions.rs
Original file line number Diff line number Diff line change
Expand Up @@ -17,23 +17,22 @@

//! Simplify expressions optimizer rule

use crate::error::DataFusionError;
use crate::execution::context::ExecutionProps;
use crate::logical_plan::ExprSchemable;
use crate::logical_plan::{
lit, DFSchema, DFSchemaRef, Expr, ExprRewritable, ExprRewriter, ExprSimplifiable,
LogicalPlan, RewriteRecursion, SimplifyInfo,
};
use crate::optimizer::optimizer::OptimizerConfig;
use crate::optimizer::optimizer::OptimizerRule;
use crate::logical_plan::{ExprSimplifiable, SimplifyInfo};
use crate::optimizer::optimizer::{OptimizerConfig, OptimizerRule};
use crate::physical_plan::planner::create_physical_expr;
use crate::scalar::ScalarValue;
use crate::{error::Result, logical_plan::Operator};
use arrow::array::new_null_array;
use arrow::datatypes::{DataType, Field, Schema};
use arrow::record_batch::RecordBatch;
use datafusion_expr::utils::from_plan;
use datafusion_expr::Volatility;
use datafusion_common::{DFSchema, DFSchemaRef, DataFusionError, Result, ScalarValue};
use datafusion_expr::{
expr_rewriter::RewriteRecursion,
expr_rewriter::{ExprRewritable, ExprRewriter},
lit,
logical_plan::LogicalPlan,
utils::from_plan,
Expr, ExprSchemable, Operator, Volatility,
};

/// Provides simplification information based on schema and properties
pub(crate) struct SimplifyContext<'a, 'b> {
Expand Down Expand Up @@ -748,22 +747,22 @@ impl<'a, S: SimplifyInfo> ExprRewriter for Simplifier<'a, S> {

#[cfg(test)]
mod tests {
use std::collections::HashMap;
use std::sync::Arc;

use arrow::array::{ArrayRef, Int32Array};
use chrono::{DateTime, TimeZone, Utc};
use datafusion_expr::{BuiltinScalarFunction, ExprSchemable};

use super::*;
use crate::assert_contains;
use crate::logical_plan::{
and, binary_expr, call_fn, col, create_udf, lit, lit_timestamp_nano, DFField,
Expr, LogicalPlanBuilder,
};
use crate::logical_plan::{call_fn, create_udf};
use crate::physical_plan::functions::make_scalar_function;
use crate::physical_plan::udf::ScalarUDF;
use crate::test_util::scan_empty;
use arrow::array::{ArrayRef, Int32Array};
use chrono::{DateTime, TimeZone, Utc};
use datafusion_common::DFField;
use datafusion_expr::{
and, binary_expr, col, lit, lit_timestamp_nano,
logical_plan::builder::LogicalPlanBuilder, BuiltinScalarFunction, Expr,
ExprSchemable,
};
use std::collections::HashMap;
use std::sync::Arc;

#[test]
fn test_simplify_or_true() {
Expand Down
19 changes: 11 additions & 8 deletions datafusion/core/src/optimizer/single_distinct_to_groupby.rs
Original file line number Diff line number Diff line change
Expand Up @@ -17,13 +17,14 @@

//! single distinct to group by optimizer rule

use crate::error::Result;
use crate::logical_plan::plan::{Aggregate, Projection};
use crate::logical_plan::ExprSchemable;
use crate::logical_plan::{col, DFSchema, Expr, LogicalPlan};
use crate::optimizer::optimizer::OptimizerConfig;
use crate::optimizer::optimizer::OptimizerRule;
use datafusion_expr::utils::{columnize_expr, from_plan};
use crate::optimizer::optimizer::{OptimizerConfig, OptimizerRule};
use datafusion_common::{DFSchema, Result};
use datafusion_expr::{
col,
logical_plan::{Aggregate, LogicalPlan, Projection},
utils::{columnize_expr, from_plan},
Expr, ExprSchemable,
};
use hashbrown::HashSet;
use std::sync::Arc;

Expand Down Expand Up @@ -200,9 +201,11 @@ impl OptimizerRule for SingleDistinctToGroupBy {
#[cfg(test)]
mod tests {
use super::*;
use crate::logical_plan::{col, count, count_distinct, lit, max, LogicalPlanBuilder};
use crate::physical_plan::aggregates;
use crate::test::*;
use datafusion_expr::{
col, count, count_distinct, lit, logical_plan::builder::LogicalPlanBuilder, max,
};

fn assert_optimized_plan_eq(plan: &LogicalPlan, expected: &str) {
let rule = SingleDistinctToGroupBy::new();
Expand Down
Loading