Skip to content

Commit

Permalink
[Enhancement](partition) Refine some auto partition behaviours (#32737)…
Browse files Browse the repository at this point in the history
… (#33412)

fix legacy planner grammer
fix nereids planner parsing
fix cases
forbid auto range partition with null column
fix CreateTableStmt with auto partition and some partition items.
1 and 2 are about #31585
doc pr: apache/doris-website#488
  • Loading branch information
zclllyybb authored Apr 9, 2024
1 parent 97850cf commit e574b35
Show file tree
Hide file tree
Showing 46 changed files with 122 additions and 95 deletions.
3 changes: 2 additions & 1 deletion be/src/exec/tablet_info.cpp
Original file line number Diff line number Diff line change
Expand Up @@ -75,6 +75,8 @@ bool VOlapTablePartKeyComparator::operator()(const BlockRowWithIndicator lhs,
bool l_use_new = std::get<2>(lhs);
bool r_use_new = std::get<2>(rhs);

VLOG_TRACE << '\n' << l_block->dump_data() << '\n' << r_block->dump_data();

if (l_row == -1) {
return false;
} else if (r_row == -1) {
Expand All @@ -93,7 +95,6 @@ bool VOlapTablePartKeyComparator::operator()(const BlockRowWithIndicator lhs,
DCHECK(_slot_locs.size() == _param_locs.size())
<< _slot_locs.size() << ' ' << _param_locs.size();

//TODO: use template to accelerate this for older compiler.
const std::vector<uint16_t>* l_index = l_use_new ? &_param_locs : &_slot_locs;
const std::vector<uint16_t>* r_index = r_use_new ? &_param_locs : &_slot_locs;

Expand Down
6 changes: 5 additions & 1 deletion be/src/vec/sink/vrow_distribution.cpp
Original file line number Diff line number Diff line change
Expand Up @@ -29,6 +29,7 @@
#include "runtime/runtime_state.h"
#include "util/doris_metrics.h"
#include "util/thrift_rpc_helper.h"
#include "vec/columns/column.h"
#include "vec/columns/column_const.h"
#include "vec/columns/column_nullable.h"
#include "vec/columns/column_vector.h"
Expand Down Expand Up @@ -331,7 +332,10 @@ Status VRowDistribution::generate_rows_distribution(
for (int i = 0; i < func_size; ++i) {
int result_idx = -1;
RETURN_IF_ERROR(part_funcs[i]->execute(part_ctxs[i].get(), block.get(), &result_idx));
VLOG_DEBUG << "Partition-calculated block:" << block->dump_data();

VLOG_DEBUG << "Partition-calculated block:" << block->dump_data(0, 1);
DCHECK(result_idx != -1);

partition_cols_idx.push_back(result_idx);
}

Expand Down
6 changes: 3 additions & 3 deletions fe/fe-core/src/main/cup/sql_parser.cup
Original file line number Diff line number Diff line change
Expand Up @@ -3164,15 +3164,15 @@ opt_partition ::=
RESULT = new ListPartitionDesc(columns, list);
:}
/* expr range partition */
| KW_AUTO KW_PARTITION KW_BY KW_RANGE function_call_expr:fnExpr
| KW_AUTO KW_PARTITION KW_BY KW_RANGE LPAREN function_call_expr:fnExpr RPAREN
LPAREN opt_all_partition_desc_list:list RPAREN
{:
ArrayList<Expr> exprs = new ArrayList<Expr>();
exprs.add(fnExpr);
RESULT = RangePartitionDesc.createRangePartitionDesc(exprs, list);
:}
| KW_AUTO KW_PARTITION KW_BY KW_RANGE function_name:functionName LPAREN expr_list:l COMMA
KW_INTERVAL expr:v ident:u RPAREN LPAREN opt_all_partition_desc_list:list RPAREN
| KW_AUTO KW_PARTITION KW_BY KW_RANGE LPAREN function_name:functionName LPAREN expr_list:l COMMA
KW_INTERVAL expr:v ident:u RPAREN RPAREN LPAREN opt_all_partition_desc_list:list RPAREN
{:
Expr fnExpr = FunctionCallExpr.functionWithIntervalConvert(functionName.getFunction().toLowerCase(), l.get(0), v, u);
ArrayList<Expr> exprs = new ArrayList<Expr>();
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -56,7 +56,7 @@ public ListPartitionDesc(ArrayList<Expr> exprs, List<String> partitionColNames,

public static ListPartitionDesc createListPartitionDesc(ArrayList<Expr> exprs,
List<AllPartitionDesc> allPartitionDescs) throws AnalysisException {
List<String> colNames = getColNamesFromExpr(exprs, true);
List<String> colNames = getColNamesFromExpr(exprs, true, true);
return new ListPartitionDesc(exprs, colNames, allPartitionDescs);
}

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -102,7 +102,9 @@ public List<String> getPartitionColNames() {
// 1. partition by list (column) : now support one slotRef
// 2. partition by range(column/function(column)) : support slotRef and some
// special function eg: date_trunc, date_floor/ceil
public static List<String> getColNamesFromExpr(ArrayList<Expr> exprs, boolean isListPartition)
// not only for auto partition. maybe we should check for project partitiion also
public static List<String> getColNamesFromExpr(ArrayList<Expr> exprs, boolean isListPartition,
boolean isAutoPartition)
throws AnalysisException {
List<String> colNames = new ArrayList<>();
for (Expr expr : exprs) {
Expand All @@ -128,7 +130,7 @@ public static List<String> getColNamesFromExpr(ArrayList<Expr> exprs, boolean is
+ expr.toSql());
}
} else if (expr instanceof SlotRef) {
if (!colNames.isEmpty() && !isListPartition) {
if (isAutoPartition && !colNames.isEmpty() && !isListPartition) {
throw new AnalysisException(
"auto create partition only support one slotRef in expr of RANGE partition. "
+ expr.toSql());
Expand Down Expand Up @@ -208,6 +210,9 @@ public void analyze(List<ColumnDef> columnDefs, Map<String, String> otherPropert
throw new AnalysisException(
"The partition column must be NOT NULL with allow_partition_column_nullable OFF");
}
if (this instanceof RangePartitionDesc && isAutoCreatePartitions && columnDef.isAllowNull()) {
throw new AnalysisException("AUTO RANGE PARTITION doesn't support NULL column");
}
if (this instanceof RangePartitionDesc && partitionExprs != null) {
if (partitionExprs.get(0) instanceof FunctionCallExpr) {
if (!columnDef.getType().isDateType()) {
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -51,9 +51,10 @@ public RangePartitionDesc(ArrayList<Expr> exprs, List<String> partitionColNames,
this.isAutoCreatePartitions = true;
}

// for parse auto partition
public static RangePartitionDesc createRangePartitionDesc(ArrayList<Expr> exprs,
List<AllPartitionDesc> allPartitionDescs) throws AnalysisException {
List<String> colNames = getColNamesFromExpr(exprs, false);
List<String> colNames = getColNamesFromExpr(exprs, false, true);
return new RangePartitionDesc(exprs, colNames, allPartitionDescs);
}

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -2085,8 +2085,7 @@ private void createOlapTable(Database db, CreateTableStmt stmt) throws UserExcep
PartitionInfo partitionInfo = null;
Map<String, Long> partitionNameToId = Maps.newHashMap();
if (partitionDesc != null) {
PartitionDesc partDesc = partitionDesc;
for (SinglePartitionDesc desc : partDesc.getSinglePartitionDescs()) {
for (SinglePartitionDesc desc : partitionDesc.getSinglePartitionDescs()) {
long partitionId = idGeneratorBuffer.getNextId();
partitionNameToId.put(desc.getPartitionName(), partitionId);
}
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -143,6 +143,9 @@ private void validatePartitionColumn(ColumnDefinition column, ConnectContext ctx
throw new AnalysisException(
"The partition column must be NOT NULL with allow_partition_column_nullable OFF");
}
if (isAutoPartition && partitionType.equalsIgnoreCase(PartitionType.RANGE.name()) && column.isNullable()) {
throw new AnalysisException("AUTO RANGE PARTITION doesn't support NULL column");
}
}

/**
Expand Down Expand Up @@ -238,19 +241,17 @@ public PartitionDesc convertToPartitionDesc(boolean isExternal) {
}

try {
ArrayList<Expr> exprs = convertToLegacyAutoPartitionExprs(partitionList);
// here we have already extracted partitionColumns
if (partitionType.equals(PartitionType.RANGE.name())) {
if (isAutoPartition) {
partitionDesc = new RangePartitionDesc(
convertToLegacyAutoPartitionExprs(partitionList),
partitionColumns, partitionDescs);
partitionDesc = new RangePartitionDesc(exprs, partitionColumns, partitionDescs);
} else {
partitionDesc = new RangePartitionDesc(partitionColumns, partitionDescs);
}
} else {
if (isAutoPartition) {
partitionDesc = new ListPartitionDesc(
convertToLegacyAutoPartitionExprs(partitionList),
partitionColumns, partitionDescs);
partitionDesc = new ListPartitionDesc(exprs, partitionColumns, partitionDescs);
} else {
partitionDesc = new ListPartitionDesc(partitionColumns, partitionDescs);
}
Expand Down Expand Up @@ -289,4 +290,20 @@ private static List<Expr> convertToLegacyArguments(List<Expression> children) {
}
}).collect(Collectors.toList());
}

/**
* Get column names and put in partitionColumns
*/
public void extractPartitionColumns() throws AnalysisException {
if (partitionList == null) {
return;
}
ArrayList<Expr> exprs = convertToLegacyAutoPartitionExprs(partitionList);
try {
partitionColumns = PartitionDesc.getColNamesFromExpr(exprs,
partitionType.equalsIgnoreCase(PartitionType.LIST.name()), isAutoPartition);
} catch (Exception e) {
throw new AnalysisException(e.getMessage(), e.getCause());
}
}
}
Original file line number Diff line number Diff line change
Expand Up @@ -90,7 +90,7 @@ public class CreateTableInfo {
private String clusterName = null;
private List<String> clusterKeysColumnNames = null;
private List<Integer> clusterKeysColumnIds = null;
private PartitionTableInfo partitionTableInfo;
private PartitionTableInfo partitionTableInfo; // get when validate

/**
* constructor for create table
Expand Down Expand Up @@ -427,6 +427,7 @@ public void validate(ConnectContext ctx) {
});

// validate partition
partitionTableInfo.extractPartitionColumns();
partitionTableInfo.validatePartitionInfo(columnMap, properties, ctx, isEnableMergeOnWrite, isExternal);

// validate distribution descriptor
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -108,7 +108,7 @@ protected void runBeforeAll() throws Exception {
+ " event_day DATETIME NOT NULL\n"
+ ")\n"
+ "DUPLICATE KEY(event_day)\n"
+ "AUTO PARTITION BY range date_trunc(event_day, \"day\") (\n"
+ "AUTO PARTITION BY range (date_trunc(event_day, \"day\")) (\n"
+ "\tPARTITION `p20230807` values [(20230807 ), (20230808 )),\n"
+ "\tPARTITION `p20020106` values [(20020106 ), (20020107 ))\n"
+ ")\n"
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -194,7 +194,7 @@ protected void runBeforeAll() throws Exception {
+ " ) ENGINE=OLAP\n"
+ " DUPLICATE KEY(l_orderkey, l_linenumber, l_partkey, l_suppkey )\n"
+ " COMMENT 'OLAP'\n"
+ " AUTO PARTITION BY range date_trunc(`l_shipdate`, 'day') ()\n"
+ " AUTO PARTITION BY range (date_trunc(`l_shipdate`, 'day')) ()\n"
+ " DISTRIBUTED BY HASH(`l_orderkey`) BUCKETS 3\n"
+ " PROPERTIES (\n"
+ " \"replication_num\" = \"1\"\n"
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -724,7 +724,7 @@ public void testCreateTablePartitionForExternalCatalog() {
+ "distributed by hash (id) properties (\"a\"=\"b\")");
} catch (Exception e) {
Assertions.assertEquals(
"internal catalog does not support functions in 'LIST' partition",
"errCode = 2, detailMessage = auto create partition only support slotRef in list partitions. func1(`id2`, '1')",
e.getMessage());
}

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -99,7 +99,7 @@ public void testCreatePartitionRange() throws Exception {
+ " city_code VARCHAR(100)\n"
+ ")\n"
+ "DUPLICATE KEY(event_day, site_id, city_code)\n"
+ "AUTO PARTITION BY range date_trunc( event_day,'day') (\n"
+ "AUTO PARTITION BY range (date_trunc( event_day,'day')) (\n"
+ "\n"
+ ")\n"
+ "DISTRIBUTED BY HASH(event_day, site_id) BUCKETS 2\n"
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -43,7 +43,7 @@ suite("partition_mv_rewrite_dimension_1") {
) ENGINE=OLAP
DUPLICATE KEY(`o_orderkey`, `o_custkey`)
COMMENT 'OLAP'
AUTO PARTITION BY range date_trunc(`o_orderdate`, 'day') ()
auto partition by range (date_trunc(`o_orderdate`, 'day')) ()
DISTRIBUTED BY HASH(`o_orderkey`) BUCKETS 96
PROPERTIES (
"replication_allocation" = "tag.location.default: 1"
Expand Down Expand Up @@ -73,7 +73,7 @@ suite("partition_mv_rewrite_dimension_1") {
) ENGINE=OLAP
DUPLICATE KEY(l_orderkey, l_linenumber, l_partkey, l_suppkey )
COMMENT 'OLAP'
AUTO PARTITION BY range date_trunc(`l_shipdate`, 'day') ()
auto partition by range (date_trunc(`l_shipdate`, 'day')) ()
DISTRIBUTED BY HASH(`l_orderkey`) BUCKETS 96
PROPERTIES (
"replication_allocation" = "tag.location.default: 1"
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -44,7 +44,7 @@ suite("partition_mv_rewrite_dimension_2_3") {
) ENGINE=OLAP
DUPLICATE KEY(`o_orderkey`, `o_custkey`)
COMMENT 'OLAP'
AUTO PARTITION BY range date_trunc(`o_orderdate`, 'day') ()
auto partition by range (date_trunc(`o_orderdate`, 'day')) ()
DISTRIBUTED BY HASH(`o_orderkey`) BUCKETS 96
PROPERTIES (
"replication_allocation" = "tag.location.default: 1"
Expand Down Expand Up @@ -74,7 +74,7 @@ suite("partition_mv_rewrite_dimension_2_3") {
) ENGINE=OLAP
DUPLICATE KEY(l_orderkey, l_linenumber, l_partkey, l_suppkey )
COMMENT 'OLAP'
AUTO PARTITION BY range date_trunc(`l_shipdate`, 'day') ()
auto partition by range (date_trunc(`l_shipdate`, 'day')) ()
DISTRIBUTED BY HASH(`l_orderkey`) BUCKETS 96
PROPERTIES (
"replication_allocation" = "tag.location.default: 1"
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -44,7 +44,7 @@ suite("partition_mv_rewrite_dimension_2_4") {
) ENGINE=OLAP
DUPLICATE KEY(`o_orderkey`, `o_custkey`)
COMMENT 'OLAP'
AUTO PARTITION BY range date_trunc(`o_orderdate`, 'day') ()
auto partition by range (date_trunc(`o_orderdate`, 'day')) ()
DISTRIBUTED BY HASH(`o_orderkey`) BUCKETS 96
PROPERTIES (
"replication_allocation" = "tag.location.default: 1"
Expand Down Expand Up @@ -74,7 +74,7 @@ suite("partition_mv_rewrite_dimension_2_4") {
) ENGINE=OLAP
DUPLICATE KEY(l_orderkey, l_linenumber, l_partkey, l_suppkey )
COMMENT 'OLAP'
AUTO PARTITION BY range date_trunc(`l_shipdate`, 'day') ()
auto partition by range (date_trunc(`l_shipdate`, 'day')) ()
DISTRIBUTED BY HASH(`l_orderkey`) BUCKETS 96
PROPERTIES (
"replication_allocation" = "tag.location.default: 1"
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -44,7 +44,7 @@ suite("partition_mv_rewrite_dimension_2_5") {
) ENGINE=OLAP
DUPLICATE KEY(`o_orderkey`, `o_custkey`)
COMMENT 'OLAP'
AUTO PARTITION BY range date_trunc(`o_orderdate`, 'day') ()
auto partition by range (date_trunc(`o_orderdate`, 'day')) ()
DISTRIBUTED BY HASH(`o_orderkey`) BUCKETS 96
PROPERTIES (
"replication_allocation" = "tag.location.default: 1"
Expand Down Expand Up @@ -74,7 +74,7 @@ suite("partition_mv_rewrite_dimension_2_5") {
) ENGINE=OLAP
DUPLICATE KEY(l_orderkey, l_linenumber, l_partkey, l_suppkey )
COMMENT 'OLAP'
AUTO PARTITION BY range date_trunc(`l_shipdate`, 'day') ()
auto partition by range (date_trunc(`l_shipdate`, 'day')) ()
DISTRIBUTED BY HASH(`l_orderkey`) BUCKETS 96
PROPERTIES (
"replication_allocation" = "tag.location.default: 1"
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -44,7 +44,7 @@ suite("partition_mv_rewrite_dimension_2_6") {
) ENGINE=OLAP
DUPLICATE KEY(`o_orderkey`, `o_custkey`)
COMMENT 'OLAP'
AUTO PARTITION BY range date_trunc(`o_orderdate`, 'day') ()
auto partition by range (date_trunc(`o_orderdate`, 'day')) ()
DISTRIBUTED BY HASH(`o_orderkey`) BUCKETS 96
PROPERTIES (
"replication_allocation" = "tag.location.default: 1"
Expand Down Expand Up @@ -74,7 +74,7 @@ suite("partition_mv_rewrite_dimension_2_6") {
) ENGINE=OLAP
DUPLICATE KEY(l_orderkey, l_linenumber, l_partkey, l_suppkey )
COMMENT 'OLAP'
AUTO PARTITION BY range date_trunc(`l_shipdate`, 'day') ()
auto partition by range (date_trunc(`l_shipdate`, 'day')) ()
DISTRIBUTED BY HASH(`l_orderkey`) BUCKETS 96
PROPERTIES (
"replication_allocation" = "tag.location.default: 1"
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -44,7 +44,7 @@ suite("partition_mv_rewrite_dimension_2_full_join") {
) ENGINE=OLAP
DUPLICATE KEY(`o_orderkey`, `o_custkey`)
COMMENT 'OLAP'
AUTO PARTITION BY range date_trunc(`o_orderdate`, 'day') ()
auto partition by range (date_trunc(`o_orderdate`, 'day')) ()
DISTRIBUTED BY HASH(`o_orderkey`) BUCKETS 96
PROPERTIES (
"replication_allocation" = "tag.location.default: 1"
Expand Down Expand Up @@ -74,7 +74,7 @@ suite("partition_mv_rewrite_dimension_2_full_join") {
) ENGINE=OLAP
DUPLICATE KEY(l_orderkey, l_linenumber, l_partkey, l_suppkey )
COMMENT 'OLAP'
AUTO PARTITION BY range date_trunc(`l_shipdate`, 'day') ()
auto partition by range (date_trunc(`l_shipdate`, 'day')) ()
DISTRIBUTED BY HASH(`l_orderkey`) BUCKETS 96
PROPERTIES (
"replication_allocation" = "tag.location.default: 1"
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -44,7 +44,7 @@ suite("partition_mv_rewrite_dimension_2_2") {
) ENGINE=OLAP
DUPLICATE KEY(`o_orderkey`, `o_custkey`)
COMMENT 'OLAP'
AUTO PARTITION BY range date_trunc(`o_orderdate`, 'day') ()
auto partition by range (date_trunc(`o_orderdate`, 'day')) ()
DISTRIBUTED BY HASH(`o_orderkey`) BUCKETS 96
PROPERTIES (
"replication_allocation" = "tag.location.default: 1"
Expand Down Expand Up @@ -74,7 +74,7 @@ suite("partition_mv_rewrite_dimension_2_2") {
) ENGINE=OLAP
DUPLICATE KEY(l_orderkey, l_linenumber, l_partkey, l_suppkey )
COMMENT 'OLAP'
AUTO PARTITION BY range date_trunc(`l_shipdate`, 'day') ()
auto partition by range (date_trunc(`l_shipdate`, 'day')) ()
DISTRIBUTED BY HASH(`l_orderkey`) BUCKETS 96
PROPERTIES (
"replication_allocation" = "tag.location.default: 1"
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -44,7 +44,7 @@ suite("partition_mv_rewrite_dimension_2_left_anti_join") {
) ENGINE=OLAP
DUPLICATE KEY(`o_orderkey`, `o_custkey`)
COMMENT 'OLAP'
AUTO PARTITION BY range date_trunc(`o_orderdate`, 'day') ()
auto partition by range (date_trunc(`o_orderdate`, 'day')) ()
DISTRIBUTED BY HASH(`o_orderkey`) BUCKETS 96
PROPERTIES (
"replication_allocation" = "tag.location.default: 1"
Expand Down Expand Up @@ -74,7 +74,7 @@ suite("partition_mv_rewrite_dimension_2_left_anti_join") {
) ENGINE=OLAP
DUPLICATE KEY(l_orderkey, l_linenumber, l_partkey, l_suppkey )
COMMENT 'OLAP'
AUTO PARTITION BY range date_trunc(`l_shipdate`, 'day') ()
auto partition by range (date_trunc(`l_shipdate`, 'day')) ()
DISTRIBUTED BY HASH(`l_orderkey`) BUCKETS 96
PROPERTIES (
"replication_allocation" = "tag.location.default: 1"
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -44,7 +44,7 @@ suite("partition_mv_rewrite_dimension_2_1") {
) ENGINE=OLAP
DUPLICATE KEY(`o_orderkey`, `o_custkey`)
COMMENT 'OLAP'
AUTO PARTITION BY range date_trunc(`o_orderdate`, 'day') ()
auto partition by range (date_trunc(`o_orderdate`, 'day')) ()
DISTRIBUTED BY HASH(`o_orderkey`) BUCKETS 96
PROPERTIES (
"replication_allocation" = "tag.location.default: 1"
Expand Down Expand Up @@ -74,7 +74,7 @@ suite("partition_mv_rewrite_dimension_2_1") {
) ENGINE=OLAP
DUPLICATE KEY(l_orderkey, l_linenumber, l_partkey, l_suppkey )
COMMENT 'OLAP'
AUTO PARTITION BY range date_trunc(`l_shipdate`, 'day') ()
auto partition by range (date_trunc(`l_shipdate`, 'day')) ()
DISTRIBUTED BY HASH(`l_orderkey`) BUCKETS 96
PROPERTIES (
"replication_allocation" = "tag.location.default: 1"
Expand Down
Loading

0 comments on commit e574b35

Please sign in to comment.