refine large size query range

This commit is contained in:
zs0 2021-11-01 10:30:45 +08:00 committed by LINxiansheng
parent 766929ee4f
commit 9ff1baa323
27 changed files with 425 additions and 231 deletions

View File

@ -827,8 +827,8 @@ void ObExprOperatorFactory::register_expr_operators()
REG_OP_ORCL(ObExprToBinaryDouble);
REG_OP_ORCL(ObExprOracleNullif);
REG_OP_ORCL(ObExprStmtId);
// for SPM
REG_OP_ORCL(ObExprEstimateNdv);
//for SPM
REG_OP_ORCL(ObExprSpmLoadPlans);
REG_OP_ORCL(ObExprSpmAlterBaseline);
REG_OP_ORCL(ObExprSpmDropBaseline);

View File

@ -146,6 +146,7 @@ int ObMergeJoin::inner_open(ObExecContext& exec_ctx) const
join_ctx->set_tenant_id(session->get_effective_tenant_id());
join_ctx->right_cache_row_buf_.projector_ = const_cast<int32_t*>(right_op_->get_projector());
join_ctx->right_cache_row_buf_.projector_size_ = right_op_->get_projector_size();
LOG_TRACE("merge join left unique", K(id_), K(is_left_unique_));
}
return ret;
}

View File

@ -103,6 +103,8 @@ int ObMergeJoinOp::inner_open()
} else if (OB_FAIL(left_fetcher_.init(*left_, ctx_.get_allocator(), &left_row_joined_)) ||
OB_FAIL(right_fetcher_.init(*right_, ctx_.get_allocator(), NULL))) {
LOG_WARN("init row fetcher failed", K(ret));
} else {
LOG_TRACE("merge join left unique", K(MY_SPEC.id_), K(MY_SPEC.is_left_unique_));
}
return ret;
}

View File

@ -192,7 +192,7 @@ int ObSQLMockSchemaUtils::mock_rowid_index(
rowid_idx_schema.set_tablegroup_id(OB_INVALID_ID);
rowid_idx_schema.set_table_type(USER_INDEX);
rowid_idx_schema.set_index_type(INDEX_TYPE_UNIQUE_LOCAL);
rowid_idx_schema.set_index_using_type(USING_BTREE);
rowid_idx_schema.set_index_using_type(USING_HASH); //set as USING_HASH, can not offer ordering.
rowid_idx_schema.set_load_type(TABLE_LOAD_TYPE_IN_DISK);
rowid_idx_schema.set_def_type(TABLE_DEF_TYPE_USER);
rowid_idx_schema.set_rowkey_column_num(base_table_schema->get_rowkey_column_num() + 1);

View File

@ -425,20 +425,19 @@ int ObFdItemFactory::do_deduce_fd_item_set(const EqualSets& equal_sets, ObIArray
ObRawExprSet* parent_exprs_ptr = NULL;
if (OB_FAIL(ret) || 1 == const_parent_exprs.count()) {
/*do nothing*/
} else if (OB_FAIL(get_parent_exprs_ptr(const_parent_exprs.at(0), parent_exprs_ptr))) {
LOG_WARN("failed to get parent exprs ptr", K(ret));
} else if (OB_FAIL(fd_item->get_parent_exprs()->assign(*parent_exprs_ptr))) {
LOG_WARN("failed to assign expr set", K(ret));
} else {
for (int64_t j = 1; OB_SUCC(ret) && j < const_parent_exprs.count(); j++) {
for (int64_t j = 0; OB_SUCC(ret) && j < const_parent_exprs.count(); j++) {
if (OB_FAIL(get_parent_exprs_ptr(const_parent_exprs.at(j), parent_exprs_ptr))) {
LOG_WARN("failed to get parent exprs ptr", K(ret));
} else if (OB_FAIL(copy_fd_item(new_fd_item, *fd_item))) {
LOG_WARN("failed to copy fd item", K(ret));
} else if (OB_FAIL(new_fd_item->get_parent_exprs()->assign(*parent_exprs_ptr))) {
LOG_WARN("failed to assign expr set", K(ret));
} else if (0 == j) {
new_fd_item->set_parent_exprs(parent_exprs_ptr);
fd_item_set.at(i) = new_fd_item;
} else if (new_fd_items.push_back(new_fd_item)) {
LOG_WARN("failed to push back fd item", K(ret));
} else {
new_fd_item->set_parent_exprs(parent_exprs_ptr);
}
}
}
@ -448,10 +447,14 @@ int ObFdItemFactory::do_deduce_fd_item_set(const EqualSets& equal_sets, ObIArray
}
} else if (final_deduce && !const_parent_exprs.empty()) {
ObRawExprSet* parent_exprs_ptr = NULL;
ObFdItem *new_fd_item = NULL;
if (OB_FAIL(get_parent_exprs_ptr(other_parent_exprs, parent_exprs_ptr))) {
LOG_WARN("failed to get parent exprs ptr", K(ret));
} else if (OB_FAIL(fd_item->get_parent_exprs()->assign(*parent_exprs_ptr))) {
LOG_WARN("failed to assign expr set", K(ret));
} else if (OB_FAIL(copy_fd_item(new_fd_item, *fd_item))) {
LOG_WARN("failed to copy fd item", K(ret));
} else {
new_fd_item->set_parent_exprs(parent_exprs_ptr);
fd_item_set.at(i) = new_fd_item;
}
} else { /*do nothing*/
}

View File

@ -6030,17 +6030,19 @@ int ObJoinOrder::create_and_add_nl_path(const sql::Path* left_path, const sql::P
LOG_WARN("failed to alloc a join path", K(ret));
} else {
join_path = new (join_path) JoinPath(this, left_path, right_path, NESTED_LOOP_JOIN, join_type, need_mat);
join_path->set_interesting_order_info(left_path->get_interesting_order_info());
if (OB_FAIL(append(join_path->ordering_, left_path->ordering_))) {
LOG_WARN("failed to append exprs", K(ret));
} else if (OB_FAIL(check_join_interesting_order(join_path))) {
LOG_WARN("failed to update join interesting order info", K(ret));
if (CONNECT_BY_JOIN != join_type) {
join_path->set_interesting_order_info(left_path->get_interesting_order_info());
if (OB_FAIL(append(join_path->ordering_, left_path->ordering_))) {
LOG_WARN("failed to append exprs", K(ret));
} else if (OB_FAIL(check_join_interesting_order(join_path))) {
LOG_WARN("failed to update join interesting order info", K(ret));
}
}
if (OB_FAIL(ret)) {
} else if (OB_FAIL(set_nl_filters(join_path, right_path, join_type, on_condition, where_condition))) {
LOG_WARN("failed to remove filters", K(ret));
} else if (CONNECT_BY_JOIN == join_type && OB_FAIL(push_down_order_siblings(join_path, right_path))) {
LOG_WARN("push down order siblings by condition failed", K(ret));
}
if (OB_FAIL(ret)) {
} else if (OB_FAIL(join_path->estimate_cost())) {
LOG_WARN("failed to calculate cost in create_nl_path", K(ret));
} else if (OB_FAIL(add_path(join_path))) {
@ -6708,15 +6710,15 @@ int ObJoinOrder::compute_one_row_info_for_table_scan(ObIArray<AccessPath*>& acce
AccessPath* access_path = NULL;
is_at_most_one_row_ = false;
for (int64_t i = 0; OB_SUCC(ret) && !is_at_most_one_row_ && i < access_paths.count(); i++) {
bool is_get = false;
bool is_one_row = false;
if (OB_ISNULL(access_path = access_paths.at(i)) || OB_ISNULL(access_path->pre_query_range_)) {
ret = OB_ERR_UNEXPECTED;
LOG_WARN("get unexpected null", K(ret));
} else if (access_path->is_inner_path_) {
/*do nothing*/
} else if (OB_FAIL(access_path->pre_query_range_->is_get(is_get))) {
LOG_WARN("failed to check if query range is get", K(ret));
} else if (is_get && (1 == access_path->est_cost_info_.ranges_.count())) {
} else if (OB_FAIL(access_path->pre_query_range_->is_at_most_one_row(is_one_row))) {
LOG_WARN("failed to check if is at most one row", K(ret));
} else if (is_one_row && (1 == access_path->est_cost_info_.ranges_.count())) {
is_at_most_one_row_ = true;
} else { /*do nothing*/
}

View File

@ -627,12 +627,13 @@ int ObLogDelUpd::check_multi_table_dml_for_px(AllocExchContext& ctx, ObShardingI
ObShardingInfo& sharding_info, const ObPhyTableLocationInfo* phy_table_locaion_info, bool& is_needed)
{
int ret = OB_SUCCESS;
if (OB_ISNULL(phy_table_locaion_info)) {
ObLogicalOperator *child = NULL;
if (OB_ISNULL(phy_table_locaion_info) || OB_ISNULL(child = get_child(first_child))) {
ret = OB_ERR_UNEXPECTED;
LOG_WARN("phy_table_locaion_info is null", K(ret));
LOG_WARN("phy_table_locaion_info is null", K(ret), K(phy_table_locaion_info), K(child));
} else if (phy_table_locaion_info->get_partition_cnt() > 1) {
LOG_TRACE("multi partition for px dml");
if (ctx.exchange_allocated_) {
if (ctx.exchange_allocated_ || NULL == child->get_sharding_info().get_phy_table_location_info()) {
is_needed = true;
} else {
is_needed = false;

View File

@ -1278,23 +1278,6 @@ int ObLogGroupBy::inner_append_not_produced_exprs(ObRawExprUniqueSet& raw_exprs)
return ret;
}
int ObLogGroupBy::child_has_exchange(const ObLogicalOperator* op, bool& find)
{
int ret = OB_SUCCESS;
if (OB_ISNULL(op) || find) {
/*do nothing*/
} else if (log_op_def::LOG_EXCHANGE == op->get_type()) {
find = true;
} else {
for (int i = 0; i < op->get_num_of_child() && OB_SUCC(ret); ++i) {
if (OB_FAIL(SMART_CALL(child_has_exchange(op->get_child(i), find)))) {
LOG_WARN("fail to find tsc recursive", K(ret));
}
}
}
return ret;
}
int ObLogGroupBy::compute_one_row_info()
{
int ret = OB_SUCCESS;

View File

@ -170,7 +170,6 @@ private:
virtual int inner_append_not_produced_exprs(ObRawExprUniqueSet& raw_exprs) const override;
int create_fd_item_from_select_list(ObFdItemSet* fd_item_set);
int child_has_exchange(const ObLogicalOperator* op, bool& find);
virtual int compute_one_row_info() override;
private:

View File

@ -51,7 +51,7 @@ int ObLogLimit::allocate_exchange_post(AllocExchContext* ctx)
{
int ret = OB_SUCCESS;
bool is_basic = false;
bool should_push_limit = (!is_calc_found_rows_ && (limit_percent_ == NULL));
bool should_push_limit = (!is_calc_found_rows_ && limit_count_ != NULL);
ObLogicalOperator* exchange_point = NULL;
ObExchangeInfo exch_info;
if (OB_ISNULL(ctx)) {

View File

@ -5744,6 +5744,8 @@ int ObLogPlan::extract_onetime_exprs(
} else if (is_onetime_expr) {
if (OB_FAIL(extract_subquery_ids(expr, idxs))) {
LOG_WARN("fail to extract param from raw expr", K(ret));
} else if (0 <= ObOptimizerUtil::find_exec_param(onetime_exprs, expr)) {
/* expr has added */
} else {
int64_t param_num = ObOptimizerUtil::find_exec_param(get_onetime_exprs(), expr);
if (param_num >= 0) {
@ -6050,8 +6052,6 @@ int ObLogPlan::generate_subplan_filter_info(const ObIArray<ObRawExpr*>& subquery
} else {
ObSEArray<SubPlanInfo*, 4> subplan_infos;
ObSEArray<SubPlanInfo*, 4> temp_subplan_infos;
ObBitSet<> temp_onetime_idxs;
ObSEArray<std::pair<int64_t, ObRawExpr*>, 4> temp_onetime_exprs;
for (int64_t i = 0; OB_SUCC(ret) && i < subquery_exprs.count(); i++) {
ObRawExpr* temp_expr = NULL;
temp_subplan_infos.reuse();
@ -6095,20 +6095,12 @@ int ObLogPlan::generate_subplan_filter_info(const ObIArray<ObRawExpr*>& subquery
}
if (OB_SUCC(ret) && !subquery_ops.empty()) {
for (int64_t i = 0; OB_SUCC(ret) && i < subquery_exprs.count(); i++) {
temp_onetime_exprs.reuse();
temp_onetime_idxs.reuse();
if (OB_ISNULL(subquery_exprs.at(i))) {
ret = OB_ERR_UNEXPECTED;
LOG_WARN("get unexpected null", K(ret));
} else if (OB_FAIL(extract_onetime_exprs(subquery_exprs.at(i), temp_onetime_exprs, temp_onetime_idxs))) {
} else if (OB_FAIL(extract_onetime_exprs(subquery_exprs.at(i), onetime_exprs, onetime_idxs))) {
LOG_WARN("failed to extract onetime exprs", K(ret));
} else if (OB_FAIL(append(onetime_exprs, temp_onetime_exprs))) {
LOG_WARN("failed to append onetime exprs", K(ret));
} else if (OB_FAIL(onetime_idxs.add_members(temp_onetime_idxs))) {
LOG_WARN("failed to add member", K(ret));
} else {
LOG_TRACE("succeed to get onetime exprs", K(*subquery_exprs.at(i)), K(temp_onetime_idxs));
}
} else { /*do nothing*/ }
}
}
if (OB_SUCC(ret)) {

View File

@ -499,6 +499,7 @@ int ObLogSet::check_if_match_partition_wise(bool& is_match)
{
int ret = OB_SUCCESS;
is_match = false;
bool find_exchange = false;
const int64_t num_of_child = get_num_of_child();
if (num_of_child < 2) {
ret = OB_ERR_UNEXPECTED;
@ -506,13 +507,19 @@ int ObLogSet::check_if_match_partition_wise(bool& is_match)
} else if (OB_ISNULL(get_child(first_child))) {
ret = OB_ERR_UNEXPECTED;
LOG_WARN("get unexpected null", K(get_child(first_child)), K(ret));
} else if (OB_FAIL(child_has_exchange(get_child(first_child), find_exchange))) {
LOG_WARN("failed to check contain exchange below", K(ret));
} else {
is_match = true;
is_match = !find_exchange;
const ObShardingInfo& sharding_info = get_child(first_child)->get_sharding_info();
for (int64_t i = 1; OB_SUCC(ret) && is_match && i < num_of_child; ++i) {
if (OB_ISNULL(get_child(i))) {
ret = OB_ERR_UNEXPECTED;
LOG_WARN("get unexpected null", K(get_child(i)), K(ret));
} else if (OB_FAIL(child_has_exchange(get_child(i), find_exchange))) {
LOG_WARN("fail to find exchange");
} else if (find_exchange) {
is_match = false;
} else if (OB_FAIL(ObShardingInfo::is_physically_equal_partitioned(
sharding_info, get_child(i)->get_sharding_info(), is_match))) {
LOG_WARN("failed to check is physically equal partitioned", K(ret));

View File

@ -127,9 +127,7 @@ int ObLogWindowFunction::allocate_exchange_post(AllocExchContext* ctx)
LOG_WARN("fail to check match parallel condition", K(ret));
} else if (can_parallel) {
is_parallel_ = true;
if (OB_FAIL(sharding_info_.copy_with_part_keys(child->get_sharding_info()))) {
LOG_WARN("failed to deep copy sharding info from child", K(ret));
}
sharding_info_.set_location_type(OB_TBL_LOCATION_DISTRIBUTED);
} else {
exch_info.dist_method_ = ObPQDistributeMethod::MAX_VALUE;
sharding_info_.set_location_type(OB_TBL_LOCATION_LOCAL);

View File

@ -6444,7 +6444,8 @@ int ObLogicalOperator::push_down_limit(AllocExchContext* ctx, ObRawExpr* limit_c
LOG_WARN("get unexpected null", K(exchange_point), K(ret));
} else if ((log_op_def::instance_of_log_table_scan(child->get_type())) &&
!is_virtual_table(static_cast<ObLogTableScan*>(child)->get_ref_table_id()) &&
NULL == static_cast<ObLogTableScan*>(child)->get_limit_expr()) {
NULL == static_cast<ObLogTableScan*>(child)->get_limit_expr() &&
!is_fetch_with_ties) {
// Do NOT allocate LIMIT operator, and push down limit onto table scan directly.
ObLogTableScan *table_scan = static_cast<ObLogTableScan *>(child);
table_scan->set_limit_offset(new_limit_count_expr, NULL);
@ -8615,3 +8616,21 @@ int ObLogicalOperator::check_subplan_filter_child_exchange_rescanable()
}
return ret;
}
int ObLogicalOperator::child_has_exchange(const ObLogicalOperator *op, bool &find)
{
int ret = OB_SUCCESS;
if (OB_ISNULL(op) || find) {
/*do nothing*/
} else if (log_op_def::LOG_EXCHANGE == op->get_type()) {
find = true;
} else {
for (int i = 0; OB_SUCC(ret) && !find && i < op->get_num_of_child(); ++i) {
if (OB_FAIL(SMART_CALL(child_has_exchange(op->get_child(i), find)))) {
LOG_WARN("fail to find tsc recursive", K(ret));
}
}
}
return ret;
}

View File

@ -1987,7 +1987,8 @@ public:
ObRawExpr*& part_expr, ObRawExpr*& subpart_expr);
int check_fulfill_cut_ratio_condition(int64_t dop, double ndv, bool& is_fulfill);
// check child operator contain exchange
int child_has_exchange(const ObLogicalOperator *op, bool &find);
public:
/* child operators */
ObSEArray<ObLogicalOperator*, 16, common::ModulePageAllocator, true> child_;

View File

@ -5373,8 +5373,6 @@ int ObOptimizerUtil::check_pushdown_filter_for_set(ObSelectStmt& parent_stmt, Ob
K(child_select_list.count()),
K(parent_select_list.count()),
K(ret));
} else if (OB_FAIL(ObTransformUtils::replace_exprs(parent_select_list, child_select_list, pushdown_filters))) {
LOG_WARN("failed to replace expr", K(ret));
}
for (int64_t i = 0; OB_SUCC(ret) && i < pushdown_filters.count(); ++i) {
ObSEArray<ObRawExpr*, 4> view_column_exprs;
@ -5382,6 +5380,10 @@ int ObOptimizerUtil::check_pushdown_filter_for_set(ObSelectStmt& parent_stmt, Ob
if (OB_ISNULL(expr)) {
ret = OB_ERR_UNEXPECTED;
LOG_WARN("unexpect null expr", K(ret));
} else if (OB_FAIL(ObTransformUtils::replace_expr(parent_select_list, child_select_list, expr))) {
SQL_LOG(WARN, "failed to replace expr", K(ret));
} else if (OB_FAIL(expr->extract_info())) {
LOG_WARN("failed to extract info", K(ret), K(*expr));
} else if (expr->has_flag(CNT_WINDOW_FUNC) || expr->has_flag(CNT_AGG) || expr->has_flag(CNT_SUB_QUERY)) {
ret = remain_filters.push_back(expr);
} else if (OB_FAIL(ObRawExprUtils::extract_column_exprs(expr, view_column_exprs))) {
@ -5391,12 +5393,16 @@ int ObOptimizerUtil::check_pushdown_filter_for_set(ObSelectStmt& parent_stmt, Ob
} else if (OB_FAIL(candi_filters.push_back(expr))) {
LOG_WARN("failed to push back predicate", K(ret));
}
if (OB_FAIL(ret)) {
} else if (OB_FAIL(ObTransformUtils::replace_expr(child_select_list, parent_select_list, expr))) {
SQL_LOG(WARN, "failed to replace expr", K(ret));
} else if (OB_FAIL(expr->extract_info())) {
LOG_WARN("failed to extract info", K(ret), K(*expr));
} else {
pushdown_filters.at(i) = expr;
}
}
if (OB_FAIL(ret)) {
// reset exprs for set
} else if (OB_FAIL(ObTransformUtils::replace_exprs(child_select_list, parent_select_list, pushdown_filters))) {
LOG_WARN("failed to replace expr", K(ret));
} else {
if (OB_SUCC(ret)) {
LOG_TRACE("success to check_pushdown_filter_for_set", K(pushdown_filters), K(candi_filters), K(remain_filters));
}
return ret;

View File

@ -337,12 +337,15 @@ int ObRawExprCanonicalizerImpl::pull_parallel_expr(ObRawExpr*& expr)
}
}
if (OB_SUCC(ret) && has_sub) {
ObOpRawExpr tmp;
ret = tmp.assign(*parent_expr); // ret will be checked in the following loop
parent_expr->clear_child();
for (int64_t i = 0; OB_SUCC(ret) && i < tmp.get_param_count(); ++i) {
ObRawExpr* sub_expr = tmp.get_param_expr(i);
if (OB_ISNULL(sub_expr)) {
ObSEArray<ObRawExpr*, 2> param_exprs;
ObRawExpr *sub_expr = NULL;
if (OB_FAIL(param_exprs.assign(parent_expr->get_param_exprs()))) {
LOG_WARN("failed to assign exprs", K(ret));
} else {
parent_expr->clear_child();
}
for (int64_t i = 0; OB_SUCC(ret) && i < param_exprs.count(); ++i) {
if (OB_ISNULL(sub_expr = param_exprs.at(i))) {
ret = OB_ERR_UNEXPECTED;
LOG_WARN("sub_expr is null", K(i));
} else if (sub_expr->get_expr_type() == parent_expr->get_expr_type()) {
@ -364,9 +367,6 @@ int ObRawExprCanonicalizerImpl::pull_parallel_expr(ObRawExpr*& expr)
}
}
}
if (OB_SUCC(ret)) {
tmp.reset();
}
}
if (OB_SUCC(ret)) {
if (expr->get_expr_type() == T_OP_AND && expr->get_param_count() == 1) {
@ -426,26 +426,24 @@ int ObRawExprCanonicalizerImpl::do_push_not(ObRawExpr*& expr)
child_expr->set_expr_type(child_expr->get_expr_type() == T_OP_AND ? T_OP_OR : T_OP_AND);
// child_expr->free_op(); @todo
expr = child_expr;
ObOpRawExpr* m_expr = static_cast<ObOpRawExpr*>(child_expr); // and, or
ObOpRawExpr tmp;
ret = tmp.assign(*m_expr); // copy old children
if (OB_SUCC(ret)) {
ObOpRawExpr *m_expr = static_cast<ObOpRawExpr *>(child_expr); // and, or
ObSEArray<ObRawExpr*, 2> param_exprs;
if (OB_FAIL(param_exprs.assign(m_expr->get_param_exprs()))) {
LOG_WARN("failed to assign exprs", K(ret));
} else {
not_expr->reset();
m_expr->clear_child();
// reuse not
not_expr->set_expr_type(T_OP_NOT);
not_expr->set_param_expr(tmp.get_param_expr(0));
not_expr->set_expr_type(T_OP_NOT); // reuse not
not_expr->set_param_expr(param_exprs.at(0));
if (OB_FAIL(not_expr->add_flag(IS_NOT))) {
LOG_WARN("failed to add flag IS_NOT", K(ret));
} else if (OB_FAIL(m_expr->add_param_expr(not_expr))) {
LOG_WARN("failed to add param expr", K(ret));
} else if (OB_FAIL(SMART_CALL(do_push_not(m_expr->get_param_expr(0))))) {
LOG_WARN("failed to do push not", K(ret));
}
}
if (OB_FAIL(ret)) {
LOG_WARN("failed to assigin expr", K(ret));
} else if (OB_FAIL(not_expr->add_flag(IS_NOT))) {
LOG_WARN("failed to add flag IS_NOT", K(ret));
} else if (OB_FAIL(m_expr->add_param_expr(not_expr))) {
LOG_WARN("failed to add param expr", K(ret));
} else if (OB_FAIL(SMART_CALL(do_push_not(m_expr->get_param_expr(0))))) {
LOG_WARN("failed to do push not", K(ret));
}
for (int64_t i = 1; OB_SUCC(ret) && i < tmp.get_param_count(); ++i) {
for (int64_t i = 1; OB_SUCC(ret) && i < param_exprs.count(); ++i) {
ObOpRawExpr* another = NULL;
if (OB_FAIL(ctx_.expr_factory_.create_raw_expr(T_OP_NOT, another))) {
LOG_WARN("create ObOpRawExpr failed", K(ret));
@ -453,7 +451,7 @@ int ObRawExprCanonicalizerImpl::do_push_not(ObRawExpr*& expr)
ret = OB_ERR_UNEXPECTED;
LOG_WARN("expr is null");
} else {
another->set_param_expr(tmp.get_param_expr(i));
another->set_param_expr(param_exprs.at(i));
if (OB_FAIL(another->add_flag(IS_NOT))) {
LOG_WARN("failed to add flag IS_NOT", K(ret));
} else if (OB_FAIL(m_expr->add_param_expr(another))) {

View File

@ -244,19 +244,18 @@ int ObQueryRange::preliminary_extract_query_range(const ColumnIArray& range_colu
}
}
}
if (OB_SUCC(ret)) {
if (OB_FAIL(normalize_range_graph(root))) {
LOG_WARN("normalize range graph failed", K(ret));
if (OB_SUCC(ret) && NULL != root) {
if (OB_FAIL(refine_large_range_graph(root))) {
LOG_WARN("failed to refine large range graph", K(ret));
} else {
SQL_REWRITE_LOG(DEBUG, "root key part", K(*root));
int64_t max_pos = -1;
table_graph_.key_part_head_ = root;
table_graph_.is_standard_range_ = is_standard_graph(root);
OZ(is_strict_equal_graph(root, 0, max_pos, table_graph_.is_equal_range_));
OZ(check_graph_type());
}
}
if (OB_SUCC(ret) && root != NULL) {
SQL_REWRITE_LOG(DEBUG, "root key part", K(*root));
int64_t max_pos = -1;
table_graph_.key_part_head_ = root;
table_graph_.is_standard_range_ = is_standard_graph(root);
OZ(is_strict_equal_graph(root, 0, max_pos, table_graph_.is_equal_range_));
OZ(check_graph_type());
}
}
if (OB_SUCC(ret)) {
if (query_range_ctx_->need_final_extact_) {
@ -338,10 +337,10 @@ int ObQueryRange::preliminary_extract_query_range(const ColumnIArray& range_colu
if (OB_SUCC(ret)) {
if (OB_FAIL(and_range_graph(and_ranges, temp_result))) {
LOG_WARN("And query range failed", K(ret));
} else if (OB_FAIL(normalize_range_graph(temp_result))) {
LOG_WARN("normalize range graph failed", K(ret));
} else if (NULL == temp_result) {
// no range left
} else if (OB_FAIL(refine_large_range_graph(temp_result))) {
LOG_WARN("failed to refine large range graph", K(ret));
} else {
int64_t max_pos = -1;
table_graph_.key_part_head_ = temp_result;
@ -369,7 +368,165 @@ int ObQueryRange::preliminary_extract_query_range(const ColumnIArray& range_colu
return ret;
}
int ObQueryRange::is_get(bool& is_range_get) const
// if the range size is large then RANGE_MAX_SIZE, remove some ranges according to pos_.offset_
int ObQueryRange::refine_large_range_graph(ObKeyPart *&key_part)
{
int ret = OB_SUCCESS;
ObSEArray<ObKeyPart*, 8> pre_key_parts;
ObSEArray<ObKeyPart*, 8> key_parts;
ObSEArray<uint64_t, 8> or_count;
ObSEArray<ObKeyPart*, 8> next_key_parts;
ObSEArray<uint64_t, 8> next_or_count;
uint64_t cur_range_size = 1;
bool need_refine = false;
if (OB_ISNULL(key_part)) {
ret = OB_ERR_UNEXPECTED;
LOG_WARN("keypart is null", K(ret), K(key_part));
} else if (OB_FAIL(key_parts.push_back(key_part)) ||
OB_FAIL(next_key_parts.push_back(key_part))) {
LOG_WARN("failed to push back", K(ret));
} else if (OB_FAIL(or_count.push_back(1))) {
LOG_WARN("failed to push back", K(ret));
}
while (OB_SUCC(ret) && !next_key_parts.empty() && !need_refine) {
if (OB_FAIL(compute_range_size(key_parts, or_count, next_key_parts, next_or_count,
cur_range_size))) {
LOG_WARN("failed to compute range size", K(ret));
} else if (cur_range_size > MAX_RANGE_SIZE) {
need_refine = true;
} else if (OB_FAIL(pre_key_parts.assign(key_parts))) {
LOG_WARN("failed to assign array", K(ret), K(key_parts));
} else if (OB_FAIL(key_parts.assign(next_key_parts))) {
LOG_WARN("failed to assign array", K(ret), K(next_key_parts));
} else if (OB_FAIL(or_count.assign(next_or_count))) {
LOG_WARN("failed to assign array", K(ret), K(next_or_count));
} else { /* do nothing */ }
}
if (OB_SUCC(ret) && need_refine) {
if (pre_key_parts.empty()) {
// first or_next_ list size is large than RANGE_MAX_SIZE, create a full key part
ObKeyPart *new_key = NULL;
if (OB_FAIL(alloc_full_key_part(new_key))) {
LOG_WARN("alloc full key part failed", K(ret));
} else if (OB_ISNULL(new_key)) {
ret = OB_ERR_UNEXPECTED;
LOG_WARN("keypart is null");
} else if (OB_FAIL(remove_precise_range_expr(0))) {
LOG_WARN("remove precise range expr failed", K(ret));
} else {
new_key->id_ = key_part->id_;
key_part = new_key;
LOG_TRACE("refine lagre query range with full key", K(cur_range_size));
}
} else if (OB_ISNULL(pre_key_parts.at(0))) {
ret = OB_ERR_UNEXPECTED;
LOG_WARN("get unexpected input", K(ret), K(pre_key_parts));
} else if (OB_FAIL(remove_precise_range_expr(pre_key_parts.at(0)->pos_.offset_ + 1))) {
LOG_WARN("remove precise range expr failed", K(ret));
} else {
// remove key part after pre key parts
LOG_TRACE("refine lagre query range remove some key parts", K(cur_range_size),
K(pre_key_parts.at(0)->pos_.offset_));
ObKeyPart *cur = NULL;;
ObKeyPart *and_next = NULL;
for (int64_t i = 0; OB_SUCC(ret) && i < pre_key_parts.count(); ++i) {
cur = pre_key_parts.at(i);
while (NULL != cur) {
if (NULL == cur->and_next_) {
cur = cur->or_next_;
} else {
and_next = cur->and_next_;
while (NULL != cur && cur->and_next_ == and_next) {
cur->and_next_ = NULL;
cur = cur->or_next_;
}
}
}
}
}
}
return ret;
}
int ObQueryRange::compute_range_size(const ObIArray<ObKeyPart*> &key_parts,
const ObIArray<uint64_t> &or_count,
ObIArray<ObKeyPart*> &next_key_parts,
ObIArray<uint64_t> &next_or_count,
uint64_t &range_size)
{
int ret = OB_SUCCESS;
next_key_parts.reuse();
next_or_count.reuse();
ObKeyPart *pre = NULL;
ObKeyPart *cur = NULL;
uint64_t count = 0;
if (OB_UNLIKELY(key_parts.empty()) || OB_UNLIKELY(key_parts.count() != or_count.count())) {
ret = OB_ERR_UNEXPECTED;
LOG_WARN("get unexpected input", K(ret), K(key_parts.count()), K(or_count.count()));
}
for (int64_t i = 0; OB_SUCC(ret) && i < key_parts.count(); ++i) {
if (OB_ISNULL(pre = key_parts.at(i))) {
ret = OB_ERR_UNEXPECTED;
LOG_WARN("get unexpected null", K(ret), K(i), K(key_parts.at(i)));
} else {
range_size -= or_count.at(i);
cur = pre->or_next_;
count = 1;
}
while (OB_SUCC(ret) && NULL != pre) {
if (NULL != cur && cur->and_next_ == pre->and_next_) {
cur = cur->or_next_;
++count;
} else if (NULL != pre->and_next_ &&
OB_FAIL(next_key_parts.push_back(pre->and_next_))) {
LOG_WARN("failed to push back", K(ret));
} else if (NULL != pre->and_next_ &&
OB_FAIL(next_or_count.push_back(or_count.at(i) * count))) {
LOG_WARN("failed to push back", K(ret));
} else {
range_size += or_count.at(i) * count;
pre = cur;
cur = NULL != cur ? cur->or_next_ : NULL;
count = 1;
}
}
}
return ret;
}
int ObQueryRange::is_at_most_one_row(bool &is_one_row) const
{
int ret = OB_SUCCESS;
is_one_row = true;
if (NULL == table_graph_.key_part_head_) {
is_one_row = false;
} else if (OB_FAIL(check_is_at_most_one_row(*table_graph_.key_part_head_, 0,
column_count_, is_one_row))) {
LOG_WARN("failed to check is get", K(ret));
}
return ret;
}
int ObQueryRange::check_is_at_most_one_row(ObKeyPart &key_part,
const int64_t depth,
const int64_t column_count,
bool &bret) const
{
int ret = OB_SUCCESS;
if (key_part.pos_.offset_ != depth
|| !key_part.is_equal_condition()
|| NULL != key_part.or_next_) {
bret = false;
} else if (NULL != key_part.and_next_) {
ret = SMART_CALL(check_is_at_most_one_row(*key_part.and_next_,
depth + 1, column_count, bret));
} else if (depth < column_count - 1) {
bret = false;
}
return ret;
}
int ObQueryRange::is_get(bool &is_range_get) const
{
return is_get(column_count_, is_range_get);
}
@ -2995,7 +3152,9 @@ int ObQueryRange::definite_in_range_graph(
{
int ret = OB_SUCCESS;
bool is_stack_overflow = false;
if (OB_FAIL(check_stack_overflow(is_stack_overflow))) {
if (OB_FAIL(THIS_WORKER.check_status())) {
LOG_WARN("check status fail", K(ret));
} else if (OB_FAIL(check_stack_overflow(is_stack_overflow))) {
LOG_WARN("failed to do stack overflow check", K(ret));
} else if (is_stack_overflow) {
ret = OB_SIZE_OVERFLOW;
@ -3012,7 +3171,8 @@ int ObQueryRange::definite_in_range_graph(
if (!root->is_equal_condition()) {
has_scan_key = true;
}
if (NULL != root->and_next_) {
if (NULL != root->and_next_ && (NULL == root->or_next_ ||
root->or_next_->and_next_ != root->and_next_)) {
if (OB_FAIL(SMART_CALL(definite_in_range_graph(params, root->and_next_, has_scan_key, dtc_params)))) {
LOG_WARN("definite and_next_ key part failed", K(ret));
}
@ -3428,7 +3588,9 @@ int ObQueryRange::and_first_search(ObSearchState& search_state, ObKeyPart* cur,
{
int ret = OB_SUCCESS;
bool is_stack_overflow = false;
if (OB_FAIL(check_stack_overflow(is_stack_overflow))) {
if (OB_UNLIKELY(THIS_WORKER.check_status())) {
LOG_WARN("check status fail", K(ret));
} else if (OB_FAIL(check_stack_overflow(is_stack_overflow))) {
LOG_WARN("failed to do stack overflow check", K(ret));
} else if (is_stack_overflow) {
ret = OB_SIZE_OVERFLOW;
@ -3880,6 +4042,8 @@ int ObQueryRange::get_tablet_ranges(
ObQueryRangeArray& ranges, ObGetMethodArray& get_methods, const ObDataTypeCastParams& dtc_params)
{
int ret = OB_SUCCESS;
int64_t last_mem_usage = allocator_.total();
int64_t query_range_mem_usage = 0;
ObSearchState search_state(allocator_);
ranges.reset();
@ -3940,7 +4104,10 @@ int ObQueryRange::get_tablet_ranges(
}
}
if (OB_SUCC(ret)) {
SQL_REWRITE_LOG(DEBUG, "get range success", K(ranges));
query_range_mem_usage = allocator_.total() - last_mem_usage;
LOG_TRACE("[SQL MEM USAGE] query range memory usage", K(query_range_mem_usage),
K(last_mem_usage));
LOG_TRACE("get range success", K(ranges));
if (table_graph_.is_equal_range_) {
search_state.range_set_.destroy();
}
@ -4839,8 +5006,10 @@ int ObQueryRange::is_strict_equal_graph(
} else {
// and direction
if (NULL != node->and_next_) {
OZ(SMART_CALL(is_strict_equal_graph(node->and_next_, cur_pos + 1, max_pos, is_strict_equal)));
} else { // check alignment
if (NULL == node->or_next_ || node->or_next_->and_next_ != node->and_next_) {
OZ(SMART_CALL(is_strict_equal_graph(node->and_next_, cur_pos + 1, max_pos, is_strict_equal)));
}
} else { // check alignment
if (-1 == max_pos) {
max_pos = cur_pos;
} else if (cur_pos != max_pos) {
@ -5009,23 +5178,6 @@ bool ObQueryRange::has_scan_key(const ObKeyPart& keypart) const
return bret;
}
int ObQueryRange::normalize_range_graph(ObKeyPart*& keypart)
{
int ret = OB_SUCCESS;
ObKeyPart* full_key = NULL;
if (keypart != NULL && keypart->pos_.offset_ > 0) {
if (OB_FAIL(alloc_full_key_part(full_key))) {
LOG_WARN("alloc full key part failed", K(ret));
} else if (OB_ISNULL(full_key)) {
ret = OB_ERR_UNEXPECTED;
LOG_WARN("keypart is null");
} else {
full_key->id_ = keypart->id_;
}
}
return ret;
}
void ObQueryRange::check_like_range_precise(
const ObString& pattern_str, const char* max_str_buf, const size_t max_str_len, const char escape)
{

View File

@ -291,8 +291,12 @@ public:
}
return bret;
}
// XXX: This function may raise problem because of reverse index.
int is_min_to_max_range(bool& is_min_to_max_range, const ObDataTypeCastParams& dtc_params);
int is_at_most_one_row(bool &is_one_row) const;
int check_is_at_most_one_row(ObKeyPart &key_part, const int64_t depth, const int64_t column_count,
bool &bret) const;
int is_get(bool& is_get) const;
int is_get(int64_t column_count, bool& is_get) const;
bool is_precise_get() const
@ -442,16 +446,18 @@ private:
bool has_scan_key(const ObKeyPart& keypart) const;
bool is_min_range_value(const common::ObObj& obj) const;
bool is_max_range_value(const common::ObObj& obj) const;
int normalize_range_graph(ObKeyPart*& keypart);
int check_is_get(ObKeyPart& key_part, const int64_t depth, const int64_t column_count, bool& bret) const;
void check_like_range_precise(
const ObString& pattern_str, const char* min_str_buf, const size_t min_str_len, const char escape);
int cast_like_obj_if_needed(const ObObj& string_obj, ObObj& buf_obj, const ObObj*& obj_ptr, ObKeyPart& out_key_part,
const ObDataTypeCastParams& dtc_params);
int refine_large_range_graph(ObKeyPart *&key_part);
int compute_range_size(const ObIArray<ObKeyPart*> &key_parts, const ObIArray<uint64_t> &or_count,
ObIArray<ObKeyPart*> &next_key_parts, ObIArray<uint64_t> &next_or_count, uint64_t &range_size);
private:
static const int64_t COMMON_KEY_PART_NUM = 256;
static const int64_t RANGE_BUCKET_SIZE = 1000;
static const int64_t MAX_RANGE_SIZE = 10000;
typedef common::ObObjStore<ObKeyPart*, common::ObIAllocator&> KeyPartStore;
private:

View File

@ -106,15 +106,14 @@ enum TRANSFORM_TYPE {
OR_EXPANSION = 1 << 13,
WIN_MAGIC = 1 << 14,
JOIN_ELIMINATION = 1 << 15,
JOIN_AGGREGATION = 1 << 16,
GROUPBY_PLACEMENT = 1 << 17,
SUBQUERY_COALESCE = 1 << 18,
WIN_GROUPBY = 1 << 19,
PREDICATE_MOVE_AROUND = 1 << 20,
NL_FULL_OUTER_JOIN = 1 << 21,
SEMI_TO_INNER = 1 << 22,
OUTERJOIN_LIMIT_PUSHDOWN = 1 << 23,
TRANSFORM_TYPE_COUNT_PLUS_ONE = 1 << 24
GROUPBY_PLACEMENT = 1 << 16,
SUBQUERY_COALESCE = 1 << 17,
WIN_GROUPBY = 1 << 18,
PREDICATE_MOVE_AROUND = 1 << 19,
NL_FULL_OUTER_JOIN = 1 << 20,
SEMI_TO_INNER = 1 << 21,
OUTERJOIN_LIMIT_PUSHDOWN = 1 << 22,
TRANSFORM_TYPE_COUNT_PLUS_ONE = 1 << 23
};
struct ObParentDMLStmt {
@ -134,8 +133,8 @@ public:
static const uint64_t ALL_TRANSFORM_RULES = TRANSFORM_TYPE_COUNT_PLUS_ONE - 1;
static const uint64_t ALL_HEURISTICS_RULES = SIMPLIFY | ANYALL | AGGR | ELIMINATE_OJ | VIEW_MERGE | WHERE_SQ_PULL_UP |
QUERY_PUSH_DOWN | SET_OP | PROJECTION_PRUNING | JOIN_ELIMINATION |
JOIN_AGGREGATION | WIN_GROUPBY | PREDICATE_MOVE_AROUND |
NL_FULL_OUTER_JOIN | OUTERJOIN_LIMIT_PUSHDOWN;
WIN_GROUPBY | PREDICATE_MOVE_AROUND | NL_FULL_OUTER_JOIN |
OUTERJOIN_LIMIT_PUSHDOWN;
ObTransformRule(ObTransformerCtx* ctx, TransMethod transform_method)
: ctx_(ctx),
transform_method_(transform_method),

View File

@ -459,9 +459,11 @@ int ObTransformWinGroupBy::check_outer_stmt_validity(WinGroupByHelper& helper, b
{
int ret = OB_SUCCESS;
is_valid = false;
if (OB_ISNULL(helper.outer_stmt_) || OB_ISNULL(helper.inner_stmt_)) {
if (OB_ISNULL(helper.outer_stmt_) || OB_ISNULL(helper.inner_stmt_)
|| OB_ISNULL(helper.outer_table_item_)) {
ret = OB_ERR_UNEXPECTED;
LOG_WARN("get unexpected null", K(helper.outer_stmt_), K(helper.inner_stmt_), K(ret));
LOG_WARN("get unexpected null", K(helper.outer_stmt_), K(helper.inner_stmt_), K(helper.outer_table_item_),
K(ret));
} else if (helper.outer_stmt_->get_group_exprs().empty() || !helper.outer_stmt_->get_condition_exprs().empty() ||
!helper.outer_stmt_->get_window_func_exprs().empty() ||
!helper.outer_stmt_->get_subquery_exprs().empty() || helper.outer_stmt_->has_sequence() ||
@ -484,14 +486,26 @@ int ObTransformWinGroupBy::check_outer_stmt_validity(WinGroupByHelper& helper, b
}
}
if (OB_SUCC(ret) && is_valid) {
ObSEArray<ObRawExpr*, 16> all_column_exprs;
ObSEArray<ObRawExpr*, 16> aggr_exprs;
ObSEArray<ObRawExpr*, 16> groupby_exprs;
ObSEArray<ObRawExpr*, 16> column_exprs;
if (OB_FAIL(append(aggr_exprs, helper.outer_stmt_->get_aggr_items()))) {
if (OB_FAIL(helper.outer_stmt_->get_column_exprs(helper.outer_table_item_->table_id_,
all_column_exprs))) {
LOG_WARN("failed to get column exprs", K(ret));
} else if (OB_FAIL(append(aggr_exprs, helper.outer_stmt_->get_aggr_items()))) {
LOG_WARN("failed to append exprs", K(ret));
} else if (OB_FAIL(ObRawExprUtils::extract_column_exprs(aggr_exprs, column_exprs))) {
LOG_WARN("failed to extract column exprs", K(ret));
} else if (OB_FAIL(ObOptimizerUtil::intersect_exprs(all_column_exprs, column_exprs,
column_exprs))) {
LOG_WARN("failed to intersect exprs", K(ret));
} else if (OB_FAIL(ObOptimizerUtil::intersect_exprs(all_column_exprs,
helper.outer_stmt_->get_group_exprs(),
groupby_exprs))) {
LOG_WARN("failed to intersect exprs", K(ret));
} else if (OB_FAIL(ObTransformUtils::convert_column_expr_to_select_expr(
helper.outer_stmt_->get_group_exprs(), *helper.inner_stmt_, helper.ref_groupby_exprs_))) {
groupby_exprs, *helper.inner_stmt_, helper.ref_groupby_exprs_))) {
LOG_WARN("failed to convert column expr to select expr", K(ret));
} else if (OB_FAIL(ObTransformUtils::convert_column_expr_to_select_expr(
column_exprs, *helper.inner_stmt_, helper.ref_column_exprs_))) {

View File

@ -485,16 +485,23 @@ int ObTransformWinMagic::transform_child_stmt(ObDMLStmt& stmt, ObSelectStmt& sub
// add push down table
for (int64_t i = 0; OB_SUCC(ret) && i < stmt.get_table_size(); ++i) {
ObSEArray<ObDMLStmt::PartExprItem, 4> part_exprs;
const ObPartHint *part_hint = NULL;
TableItem *table = NULL;
if (!push_down_table_ids.has_member(i + 1)) {
// do nothing
} else if (OB_FAIL(subquery.get_table_items().push_back(stmt.get_table_item(i)))) {
} else if (OB_ISNULL(table = stmt.get_table_item(i))) {
ret = OB_ERR_UNEXPECTED;
LOG_WARN("unexpected null", K(ret), K(table));
} else if (OB_FAIL(subquery.get_table_items().push_back(table))) {
LOG_WARN("failed to push back table item", K(ret));
} else if (OB_FAIL(stmt.get_part_expr_items(stmt.get_table_item(i)->table_id_, part_exprs))) {
} else if (OB_FAIL(stmt.get_part_expr_items(table->table_id_, part_exprs))) {
LOG_WARN("failed to get part expr items", K(ret));
} else if (part_exprs.empty()) {
// do nothing
} else if (OB_FAIL(subquery.set_part_expr_items(part_exprs))) {
} else if (!part_exprs.empty() && OB_FAIL(subquery.set_part_expr_items(part_exprs))) {
LOG_WARN("failed to set part expr item", K(ret));
} else if (NULL == (part_hint = stmt.get_stmt_hint().get_part_hint(table->table_id_))) {
// do nothing
} else if (OB_FAIL(subquery.get_stmt_hint().part_hints_.push_back(*part_hint))) {
LOG_WARN("failed to push back hints", K(ret));
}
}
// add push down column

View File

@ -219,7 +219,7 @@ int ObTransformerImpl::transform_rule_set_in_one_iteration(
APPLY_RULE_IF_NEEDED(OR_EXPANSION, ObTransformOrExpansion);
APPLY_RULE_IF_NEEDED(WIN_MAGIC, ObTransformWinMagic);
APPLY_RULE_IF_NEEDED(WIN_GROUPBY, ObTransformWinGroupBy);
APPLY_RULE_IF_NEEDED(JOIN_AGGREGATION, ObTransformAggrSubquery);
APPLY_RULE_IF_NEEDED(AGGR_SUBQUERY, ObTransformAggrSubquery);
APPLY_RULE_IF_NEEDED(GROUPBY_PLACEMENT, ObTransformGroupByPlacement);
APPLY_RULE_IF_NEEDED(AGGR, ObTransformAggregate);
// project pruning should be done after window function transformation rules

View File

@ -8017,9 +8017,9 @@ SQL: SELECT (select max(t1.c1) from t1) as field from t1 group by field;
------------------------------------------------------------------
|0 |HASH GROUP BY | |1 |809753|
|1 | NESTED-LOOP JOIN CARTESIAN | |500000 |645625|
|2 | SUBPLAN SCAN |VIEW4 |1 |38 |
|2 | SUBPLAN SCAN |VIEW3 |1 |38 |
|3 | SCALAR GROUP BY | |1 |38 |
|4 | SUBPLAN SCAN |VIEW5 |1 |37 |
|4 | SUBPLAN SCAN |VIEW4 |1 |37 |
|5 | LIMIT | |1 |37 |
|6 | PX COORDINATOR MERGE SORT | |1 |37 |
|7 | EXCHANGE OUT DISTR |:EX10000 |1 |37 |
@ -8035,16 +8035,16 @@ SQL: SELECT (select max(t1.c1) from t1) as field from t1 group by field;
Outputs & filters:
-------------------------------------
0 - output([VIEW4.max(t1.c1)]), filter(nil),
group([VIEW4.max(t1.c1)]), agg_func(nil)
1 - output([VIEW4.max(t1.c1)]), filter(nil),
0 - output([VIEW3.max(t1.c1)]), filter(nil),
group([VIEW3.max(t1.c1)]), agg_func(nil)
1 - output([VIEW3.max(t1.c1)]), filter(nil),
conds(nil), nl_params_(nil), batch_join=false
2 - output([VIEW4.max(t1.c1)]), filter(nil),
access([VIEW4.max(t1.c1)])
3 - output([T_FUN_MAX(VIEW5.c1)]), filter(nil),
group(nil), agg_func([T_FUN_MAX(VIEW5.c1)])
4 - output([VIEW5.c1]), filter(nil),
access([VIEW5.c1])
2 - output([VIEW3.max(t1.c1)]), filter(nil),
access([VIEW3.max(t1.c1)])
3 - output([T_FUN_MAX(VIEW4.c1)]), filter(nil),
group(nil), agg_func([T_FUN_MAX(VIEW4.c1)])
4 - output([VIEW4.c1]), filter(nil),
access([VIEW4.c1])
5 - output([t1.c1]), filter(nil), limit(1), offset(nil)
6 - output([t1.c1]), filter(nil), sort_keys([t1.c1, DESC])
7 - output([t1.c1]), filter(nil), dop=1

View File

@ -9100,9 +9100,9 @@ SQL: SELECT (select max(t1.c1) from t1) as field from t1 group by field;
----------------------------------------------------------------
|0 |HASH GROUP BY | |1 |880 |
|1 | NESTED-LOOP JOIN CARTESIAN | |500 |715 |
|2 | SUBPLAN SCAN |VIEW4 |1 |38 |
|2 | SUBPLAN SCAN |VIEW3 |1 |38 |
|3 | SCALAR GROUP BY | |1 |38 |
|4 | SUBPLAN SCAN |VIEW5 |1 |37 |
|4 | SUBPLAN SCAN |VIEW4 |1 |37 |
|5 | LIMIT | |1 |37 |
|6 | PX COORDINATOR MERGE SORT | |1 |37 |
|7 | EXCHANGE OUT DISTR |:EX10000 |1 |37 |
@ -9118,16 +9118,16 @@ SQL: SELECT (select max(t1.c1) from t1) as field from t1 group by field;
Outputs & filters:
-------------------------------------
0 - output([VIEW4.max(t1.c1)]), filter(nil),
group([VIEW4.max(t1.c1)]), agg_func(nil)
1 - output([VIEW4.max(t1.c1)]), filter(nil),
0 - output([VIEW3.max(t1.c1)]), filter(nil),
group([VIEW3.max(t1.c1)]), agg_func(nil)
1 - output([VIEW3.max(t1.c1)]), filter(nil),
conds(nil), nl_params_(nil), batch_join=false
2 - output([VIEW4.max(t1.c1)]), filter(nil),
access([VIEW4.max(t1.c1)])
3 - output([T_FUN_MAX(VIEW5.c1)]), filter(nil),
group(nil), agg_func([T_FUN_MAX(VIEW5.c1)])
4 - output([VIEW5.c1]), filter(nil),
access([VIEW5.c1])
2 - output([VIEW3.max(t1.c1)]), filter(nil),
access([VIEW3.max(t1.c1)])
3 - output([T_FUN_MAX(VIEW4.c1)]), filter(nil),
group(nil), agg_func([T_FUN_MAX(VIEW4.c1)])
4 - output([VIEW4.c1]), filter(nil),
access([VIEW4.c1])
5 - output([t1.c1]), filter(nil), limit(1), offset(nil)
6 - output([t1.c1]), filter(nil), sort_keys([t1.c1, DESC])
7 - output([t1.c1]), filter(nil), dop=1
@ -26645,47 +26645,51 @@ SQL: select * from t1 where t1.c2 = 5 or exists (select 1 from t2 where t1.c1 =
==================================================================
|ID|OPERATOR |NAME |EST. ROWS|COST|
------------------------------------------------------------------
|0 |PX COORDINATOR | |750 |1893|
|1 | EXCHANGE OUT DISTR |:EX10001 |750 |1786|
|2 | UNION ALL | |750 |1786|
|0 |UNION ALL | |750 |1869|
|1 | PX COORDINATOR | |500 |389 |
|2 | EXCHANGE OUT DISTR |:EX10000 |500 |342 |
|3 | PX PARTITION ITERATOR | |500 |342 |
|4 | TABLE SCAN |t1(idx_t1_c2)|500 |342 |
|5 | MERGE JOIN | |251 |1158|
|6 | SORT | |250 |729 |
|7 | PX PARTITION ITERATOR | |250 |384 |
|8 | TABLE SCAN |t1 |250 |384 |
|9 | EXCHANGE IN MERGE SORT DISTR| |300 |206 |
|10| EXCHANGE OUT DISTR (PKEY) |:EX10000 |300 |192 |
|11| PX PARTITION ITERATOR | |300 |192 |
|12| TABLE SCAN |t2 |300 |192 |
|5 | PX COORDINATOR | |251 |1194|
|6 | EXCHANGE OUT DISTR |:EX20001 |251 |1158|
|7 | MERGE JOIN | |251 |1158|
|8 | SORT | |250 |729 |
|9 | PX PARTITION ITERATOR | |250 |384 |
|10| TABLE SCAN |t1 |250 |384 |
|11| EXCHANGE IN MERGE SORT DISTR| |300 |206 |
|12| EXCHANGE OUT DISTR (PKEY) |:EX20000 |300 |192 |
|13| PX PARTITION ITERATOR | |300 |192 |
|14| TABLE SCAN |t2 |300 |192 |
==================================================================
Outputs & filters:
-------------------------------------
0 - output([UNION([1])], [UNION([2])]), filter(nil)
1 - output([UNION([1])], [UNION([2])]), filter(nil), dop=1
2 - output([UNION([1])], [UNION([2])]), filter(nil)
1 - output([t1.c1], [t1.c2]), filter(nil)
2 - output([t1.c1], [t1.c2]), filter(nil), dop=1
3 - output([t1.c1], [t1.c2]), filter(nil),
affinitize, partition wise, force partition granule, asc.
force partition granule, asc.
4 - output([t1.c1], [t1.c2]), filter(nil),
access([t1.c1], [t1.c2]), partitions(p[0-4]),
is_index_back=false,
range_key([t1.c2], [t1.c1]), range(5,MIN ; 5,MAX),
range_cond([t1.c2 = ?])
5 - output([t1.c1], [t1.c2]), filter(nil),
equal_conds([t1.c1 = t2.c1]), other_conds(nil)
6 - output([t1.c1], [t1.c2]), filter(nil), sort_keys([t1.c1, ASC]), local merge sort
5 - output([t1.c1], [t1.c2]), filter(nil)
6 - output([t1.c1], [t1.c2]), filter(nil), dop=1
7 - output([t1.c1], [t1.c2]), filter(nil),
affinitize, partition wise, force partition granule, asc.
8 - output([t1.c1], [t1.c2]), filter([lnnvl(t1.c2 = ?)]),
equal_conds([t1.c1 = t2.c1]), other_conds(nil)
8 - output([t1.c1], [t1.c2]), filter(nil), sort_keys([t1.c1, ASC]), local merge sort
9 - output([t1.c1], [t1.c2]), filter(nil),
affinitize, force partition granule, asc.
10 - output([t1.c1], [t1.c2]), filter([lnnvl(t1.c2 = ?)]),
access([t1.c1], [t1.c2]), partitions(p[0-4]),
is_index_back=false, filter_before_indexback[false],
range_key([t1.c1]), range(MIN ; MAX)always true
9 - output([t2.c1]), filter(nil), sort_keys([t2.c1, ASC]), Local Order
10 - (#keys=1, [t2.c1]), output([t2.c1]), filter(nil), dop=1
11 - output([t2.c1]), filter(nil),
11 - output([t2.c1]), filter(nil), sort_keys([t2.c1, ASC]), Local Order
12 - (#keys=1, [t2.c1]), output([t2.c1]), filter(nil), dop=1
13 - output([t2.c1]), filter(nil),
force partition granule, asc.
12 - output([t2.c1]), filter(nil),
14 - output([t2.c1]), filter(nil),
access([t2.c1]), partitions(p[0-2]),
is_index_back=false,
range_key([t2.c1]), range(MIN ; MAX)always true

View File

@ -124,9 +124,9 @@ SQL: select * from t9 where c2 = (select max(c2) from t9);
|ID|OPERATOR |NAME |EST. ROWS|COST|
--------------------------------------------------------
|0 |NESTED-LOOP JOIN | |1 |126 |
|1 | SUBPLAN SCAN |VIEW4 |1 |37 |
|1 | SUBPLAN SCAN |VIEW3 |1 |37 |
|2 | SCALAR GROUP BY| |1 |37 |
|3 | SUBPLAN SCAN |VIEW5 |1 |37 |
|3 | SUBPLAN SCAN |VIEW4 |1 |37 |
|4 | TABLE SCAN |t9(idx_t9,Reverse)|1 |37 |
|5 | TABLE SCAN |t9(idx_t9) |1 |89 |
========================================================
@ -134,13 +134,13 @@ SQL: select * from t9 where c2 = (select max(c2) from t9);
Outputs & filters:
-------------------------------------
0 - output([t9.c1], [t9.c2], [t9.c3], [t9.c4]), filter(nil),
conds(nil), nl_params_([VIEW4.max(c2)])
1 - output([VIEW4.max(c2)]), filter(nil),
access([VIEW4.max(c2)])
2 - output([T_FUN_MAX(VIEW5.c2)]), filter(nil),
group(nil), agg_func([T_FUN_MAX(VIEW5.c2)])
3 - output([VIEW5.c2]), filter(nil),
access([VIEW5.c2])
conds(nil), nl_params_([VIEW3.max(c2)])
1 - output([VIEW3.max(c2)]), filter(nil),
access([VIEW3.max(c2)])
2 - output([T_FUN_MAX(VIEW4.c2)]), filter(nil),
group(nil), agg_func([T_FUN_MAX(VIEW4.c2)])
3 - output([VIEW4.c2]), filter(nil),
access([VIEW4.c2])
4 - output([t9.c2]), filter([(T_OP_IS_NOT, t9.c2, NULL, 0)]),
access([t9.c2]), partitions(p0),
limit(1), offset(nil)

View File

@ -4393,9 +4393,9 @@ SQL: select * from t9 where c2 = (select max(c2) from t9);
"view_base_item":-1
},
{
"table_id":-6,
"table_name":"VIEW4",
"alias_name":"VIEW4",
"table_id":-5,
"table_name":"VIEW3",
"alias_name":"VIEW3",
"synonym_name":"",
"synonym_db_name":"",
"table_type":2,
@ -4640,7 +4640,7 @@ SQL: select * from t9 where c2 = (select max(c2) from t9);
},
{
"column_id":16,
"table_id":-6,
"table_id":-5,
"column":"max(c2)",
"auto_filled_timestamp":false,
"default_value": {
@ -4675,10 +4675,10 @@ SQL: select * from t9 where c2 = (select max(c2) from t9);
"rel_id": [
2
],
"table_id":-6,
"table_id":-5,
"column_id":16,
"database_name":"",
"table_name":"VIEW4",
"table_name":"VIEW3",
"synonym_name":"",
"synonym_db_name":"",
"column_name":"max(c2)",
@ -4958,7 +4958,7 @@ SQL: select * from t9 where c2 = (select max(c2) from t9);
"is_join":false
},
{
"table_id":-6,
"table_id":-5,
"is_join":false
}
],
@ -5071,10 +5071,10 @@ SQL: select * from t9 where c2 = (select max(c2) from t9);
"rel_id": [
2
],
"table_id":-6,
"table_id":-5,
"column_id":16,
"database_name":"",
"table_name":"VIEW4",
"table_name":"VIEW3",
"synonym_name":"",
"synonym_db_name":"",
"column_name":"max(c2)",
@ -5175,9 +5175,9 @@ SQL: select * from t9 where c2 = (select max(c2) from t9);
"transpose_item":null,
"table": [
{
"table_id":-7,
"table_name":"VIEW5",
"alias_name":"VIEW5",
"table_id":-6,
"table_name":"VIEW4",
"alias_name":"VIEW4",
"synonym_name":"",
"synonym_db_name":"",
"table_type":2,
@ -5198,7 +5198,7 @@ SQL: select * from t9 where c2 = (select max(c2) from t9);
"column": [
{
"column_id":16,
"table_id":-7,
"table_id":-6,
"column":"c2",
"auto_filled_timestamp":false,
"default_value": {
@ -5233,10 +5233,10 @@ SQL: select * from t9 where c2 = (select max(c2) from t9);
"rel_id": [
1
],
"table_id":-7,
"table_id":-6,
"column_id":16,
"database_name":"",
"table_name":"VIEW5",
"table_name":"VIEW4",
"synonym_name":"",
"synonym_db_name":"",
"column_name":"c2",
@ -5315,10 +5315,10 @@ SQL: select * from t9 where c2 = (select max(c2) from t9);
"rel_id": [
1
],
"table_id":-7,
"table_id":-6,
"column_id":16,
"database_name":"",
"table_name":"VIEW5",
"table_name":"VIEW4",
"synonym_name":"",
"synonym_db_name":"",
"column_name":"c2",
@ -5370,7 +5370,7 @@ SQL: select * from t9 where c2 = (select max(c2) from t9);
"nocycle":false,
"from": [
{
"table_id":-7,
"table_id":-6,
"is_join":false
}
],
@ -5447,10 +5447,10 @@ SQL: select * from t9 where c2 = (select max(c2) from t9);
"rel_id": [
1
],
"table_id":-7,
"table_id":-6,
"column_id":16,
"database_name":"",
"table_name":"VIEW5",
"table_name":"VIEW4",
"synonym_name":"",
"synonym_db_name":"",
"column_name":"c2",