[FIX](filter) update for filter_by_select logic (#25007)

this pr is aim to update for filter_by_select logic and change delete limit

only support scala type in delete statement where condition
only support column nullable and predict column support filter_by_select logic, because we can not push down non-scala type to storage layer to pack in predict column but do filter logic
This commit is contained in:
amory
2023-10-09 21:27:40 +08:00
committed by GitHub
parent 37247ac449
commit 53b46b7e6c
23 changed files with 125 additions and 345 deletions

View File

@ -199,26 +199,6 @@ vectorized::IColumn::MutablePtr Schema::get_predicate_column_ptr(const Field& fi
case FieldType::OLAP_FIELD_TYPE_DECIMAL128I:
ptr = doris::vectorized::PredicateColumnType<TYPE_DECIMAL128I>::create();
break;
case FieldType::OLAP_FIELD_TYPE_ARRAY:
ptr = doris::vectorized::ColumnArray::create(
get_predicate_column_ptr(*field.get_sub_field(0), reader_type),
doris::vectorized::ColumnArray::ColumnOffsets::create());
break;
case FieldType::OLAP_FIELD_TYPE_STRUCT: {
size_t field_size = field.get_sub_field_count();
doris::vectorized::MutableColumns columns(field_size);
for (size_t i = 0; i < field_size; i++) {
columns[i] = get_predicate_column_ptr(*field.get_sub_field(i), reader_type);
}
ptr = doris::vectorized::ColumnStruct::create(std::move(columns));
break;
}
case FieldType::OLAP_FIELD_TYPE_MAP:
ptr = doris::vectorized::ColumnMap::create(
get_predicate_column_ptr(*field.get_sub_field(0), reader_type),
get_predicate_column_ptr(*field.get_sub_field(1), reader_type),
doris::vectorized::ColumnArray::ColumnOffsets::create());
break;
default:
LOG(FATAL) << "Unexpected type when choosing predicate column, type=" << int(field.type());
}

View File

@ -425,9 +425,13 @@ public:
* convert(convert MutablePtr to ImmutablePtr or convert ImmutablePtr to MutablePtr)
* happends in filter_by_selector because of mem-reuse logic or ColumnNullable, I think this is meaningless;
* So using raw ptr directly here.
* NOTICE: only column_nullable and predict_column, column_dictionary now support filter_by_selector
*/
virtual Status filter_by_selector(const uint16_t* sel, size_t sel_size, IColumn* col_ptr) {
LOG(FATAL) << "column not support filter_by_selector";
LOG(FATAL) << get_name()
<< " do not support filter_by_selector, only column_nullable, column_dictionary "
"and predict_column "
"support";
__builtin_unreachable();
}

View File

@ -808,38 +808,6 @@ void ColumnArray::insert_indices_from(const IColumn& src, const int* indices_beg
}
}
Status ColumnArray::filter_by_selector(const uint16_t* sel, size_t sel_size, IColumn* col_ptr) {
auto to = reinterpret_cast<vectorized::ColumnArray*>(col_ptr);
auto& to_offsets = to->get_offsets();
size_t element_size = 0;
size_t max_offset = 0;
for (size_t i = 0; i < sel_size; ++i) {
element_size += size_at(sel[i]);
max_offset = std::max(max_offset, offset_at(sel[i]));
}
if (max_offset > std::numeric_limits<uint16_t>::max()) {
return Status::Corruption("array elements too large than uint16_t::max");
}
to_offsets.reserve(to_offsets.size() + sel_size);
auto nested_sel = std::make_unique<uint16_t[]>(element_size);
size_t nested_sel_size = 0;
for (size_t i = 0; i < sel_size; ++i) {
auto row_off = offset_at(sel[i]);
auto row_size = size_at(sel[i]);
to_offsets.push_back(to_offsets.back() + row_size);
for (auto j = 0; j < row_size; ++j) {
nested_sel[nested_sel_size++] = row_off + j;
}
}
if (nested_sel_size > 0) {
return data->filter_by_selector(nested_sel.get(), nested_sel_size, &to->get_data());
}
return Status::OK();
}
ColumnPtr ColumnArray::replicate(const IColumn::Offsets& replicate_offsets) const {
if (replicate_offsets.empty()) return clone_empty();

View File

@ -249,7 +249,6 @@ public:
offsets->clear();
}
Status filter_by_selector(const uint16_t* sel, size_t sel_size, IColumn* col_ptr) override;
size_t get_number_of_dimensions() const {
const auto* nested_array = check_and_get_column<ColumnArray>(*data);
if (!nested_array) {

View File

@ -405,42 +405,6 @@ size_t ColumnMap::filter(const Filter& filter) {
return get_offsets().size();
}
Status ColumnMap::filter_by_selector(const uint16_t* sel, size_t sel_size, IColumn* col_ptr) {
auto to = reinterpret_cast<vectorized::ColumnMap*>(col_ptr);
auto& to_offsets = to->get_offsets();
size_t element_size = 0;
size_t max_offset = 0;
for (size_t i = 0; i < sel_size; ++i) {
element_size += size_at(sel[i]);
max_offset = std::max(max_offset, offset_at(sel[i]));
}
if (max_offset > std::numeric_limits<uint16_t>::max()) {
return Status::Corruption("map elements too large than uint16_t::max");
}
to_offsets.reserve(to_offsets.size() + sel_size);
auto nested_sel = std::make_unique<uint16_t[]>(element_size);
size_t nested_sel_size = 0;
for (size_t i = 0; i < sel_size; ++i) {
auto row_off = offset_at(sel[i]);
auto row_size = size_at(sel[i]);
to_offsets.push_back(to_offsets.back() + row_size);
for (auto j = 0; j < row_size; ++j) {
nested_sel[nested_sel_size++] = row_off + j;
}
}
if (nested_sel_size > 0) {
static_cast<void>(keys_column->filter_by_selector(nested_sel.get(), nested_sel_size,
&to->get_keys()));
static_cast<void>(values_column->filter_by_selector(nested_sel.get(), nested_sel_size,
&to->get_values()));
}
return Status::OK();
}
ColumnPtr ColumnMap::permute(const Permutation& perm, size_t limit) const {
// Make a temp column array
auto k_arr =

View File

@ -114,7 +114,6 @@ public:
ColumnPtr filter(const Filter& filt, ssize_t result_size_hint) const override;
size_t filter(const Filter& filter) override;
Status filter_by_selector(const uint16_t* sel, size_t sel_size, IColumn* col_ptr) override;
ColumnPtr permute(const Permutation& perm, size_t limit) const override;
ColumnPtr replicate(const Offsets& offsets) const override;
void replicate(const uint32_t* indexs, size_t target_size, IColumn& column) const override;

View File

@ -362,8 +362,16 @@ Status ColumnNullable::filter_by_selector(const uint16_t* sel, size_t sel_size,
ColumnPtr null_map_ptr = nullable_col_ptr->null_map;
RETURN_IF_ERROR(get_nested_column().filter_by_selector(
sel, sel_size, const_cast<doris::vectorized::IColumn*>(nest_col_ptr.get())));
RETURN_IF_ERROR(get_null_map_column().filter_by_selector(
sel, sel_size, const_cast<doris::vectorized::IColumn*>(null_map_ptr.get())));
//insert cur nullmap into result nullmap which is empty
auto& res_nullmap = reinterpret_cast<vectorized::ColumnVector<UInt8>*>(
const_cast<doris::vectorized::IColumn*>(null_map_ptr.get()))
->get_data();
DCHECK(res_nullmap.empty());
res_nullmap.resize(sel_size);
auto& cur_nullmap = get_null_map_column().get_data();
for (size_t i = 0; i < sel_size; i++) {
res_nullmap[i] = cur_nullmap[sel[i]];
}
return Status::OK();
}

View File

@ -262,16 +262,6 @@ size_t ColumnStruct::filter(const Filter& filter) {
return result_size;
}
Status ColumnStruct::filter_by_selector(const uint16_t* sel, size_t sel_size, IColumn* col_ptr) {
auto to = reinterpret_cast<vectorized::ColumnStruct*>(col_ptr);
const size_t tuple_size = columns.size();
DCHECK_EQ(to->tuple_size(), tuple_size);
for (size_t i = 0; i < tuple_size; ++i) {
static_cast<void>(columns[i]->filter_by_selector(sel, sel_size, &to->get_column(i)));
}
return Status::OK();
}
ColumnPtr ColumnStruct::permute(const Permutation& perm, size_t limit) const {
const size_t tuple_size = columns.size();
Columns new_columns(tuple_size);

View File

@ -150,7 +150,6 @@ public:
void insert_range_from(const IColumn& src, size_t start, size_t length) override;
ColumnPtr filter(const Filter& filt, ssize_t result_size_hint) const override;
size_t filter(const Filter& filter) override;
Status filter_by_selector(const uint16_t* sel, size_t sel_size, IColumn* col_ptr) override;
ColumnPtr permute(const Permutation& perm, size_t limit) const override;
ColumnPtr replicate(const Offsets& offsets) const override;
void replicate(const uint32_t* counts, size_t target_size, IColumn& column) const override;

View File

@ -408,12 +408,6 @@ public:
ColumnPtr filter(const IColumn::Filter& filt, ssize_t result_size_hint) const override;
size_t filter(const IColumn::Filter& filter) override;
// note(wb) this method is only used in storage layer now
Status filter_by_selector(const uint16_t* sel, size_t sel_size, IColumn* col_ptr) override {
insert_res_column(sel, sel_size, reinterpret_cast<vectorized::ColumnVector<T>*>(col_ptr));
return Status::OK();
}
ColumnPtr permute(const IColumn::Permutation& perm, size_t limit) const override;
// ColumnPtr index(const IColumn & indexes, size_t limit) const override;

View File

@ -663,6 +663,12 @@ bool VScanNode::_is_predicate_acting_on_slot(
if (_slot_id_to_value_range.end() == entry) {
return false;
}
// if the slot is a complex type(array/map/struct), we do not push down the predicate, because
// we delete pack these type into predict column, and origin pack action is wrong. we should
// make sense to push down this complex type after we delete predict column.
if (is_complex_type(remove_nullable(slot_ref->data_type()))) {
return false;
}
*slot_desc = entry->second.first;
DCHECK(child_contains_slot != nullptr);
if (child_contains_slot->type().type != (*slot_desc)->type().type ||

View File

@ -131,7 +131,7 @@ public class DeleteStmt extends DdlStmt {
try {
analyzePredicate(wherePredicate, analyzer);
checkDeleteConditions();
} catch (Exception e) {
} catch (AnalysisException e) {
if (!(((OlapTable) targetTable).getKeysType() == KeysType.UNIQUE_KEYS)) {
throw new AnalysisException(e.getMessage(), e.getCause());
}
@ -333,6 +333,14 @@ public class DeleteStmt extends DdlStmt {
}
Column column = nameToColumn.get(columnName);
// TODO(Now we can not push down non-scala type like array/map/struct to storage layer because of
// predict_column in be not support non-scala type, so we just should ban this type in delete predict, when
// we delete predict_column in be we should delete this ban)
if (!column.getType().isScalarType()) {
throw new AnalysisException(String.format("Can not apply delete condition to column type: %s ",
column.getType()));
}
// Due to rounding errors, most floating-point numbers end up being slightly imprecise,
// it also means that numbers expected to be equal often differ slightly, so we do not allow compare with
// floating-point numbers, floating-point number not allowed in where clause

View File

@ -2,4 +2,6 @@
-- !sql --
2 [12, 3]
3 []
4 \N
5 \N

View File

@ -1,4 +1,7 @@
-- This file is automatically generated. You should know what you did if you want to edit this
-- !sql --
1 {1:"a", 2:"doris"}
2 {}
3 \N
4 \N
5 \N

View File

@ -1,4 +1,7 @@
-- This file is automatically generated. You should know what you did if you want to edit this
-- !sql --
1 {1, "a"}
2 \N
3 \N
4 \N
5 \N

Binary file not shown.

File diff suppressed because one or more lines are too long

View File

@ -21,6 +21,10 @@ suite("test_array_column_delete") {
sql """ DROP TABLE IF EXISTS ${tableName}; """
sql """ CREATE TABLE IF NOT EXISTS ${tableName} (id INT NULL, c_array ARRAY<INT> NULL) ENGINE=OLAP DUPLICATE KEY(id) DISTRIBUTED BY HASH(id) BUCKETS 4 PROPERTIES ( "replication_allocation" = "tag.location.default: 1","in_memory" = "false","storage_format" = "V2") """
sql """ insert into ${tableName} values(1, NULL),(2,[12,3]),(3,[]),(4,NULL),(5,NULL) """
sql """ DELETE FROM ${tableName} WHERE c_array is NULL """
test {
sql """ DELETE FROM ${tableName} WHERE c_array is NULL """
exception("java.sql.SQLException: errCode = 2, detailMessage = errCode = 2, detailMessage = Can not apply delete condition to column type: ARRAY<INT>")
}
sql """ DELETE FROM ${tableName} WHERE id = 1; """
qt_sql """ SELECT * FROM ${tableName} order by id """
}

View File

@ -22,6 +22,10 @@ suite("test_map_column_delete") {
sql """ CREATE TABLE IF NOT EXISTS ${tableName} (id INT NULL, m_map MAP<INT, VARCHAR(30)> NULL) ENGINE=OLAP DUPLICATE KEY(id) DISTRIBUTED BY HASH(id) BUCKETS 4 PROPERTIES ( "replication_allocation" = "tag.location.default: 1","in_memory" = "false","storage_format" = "V2") """
sql """ insert into ${tableName} values(1, {1:'a', 2:"doris"}),(2,{}),(3,NULL),(4,NULL),(5,NULL) """
sql """ DELETE FROM ${tableName} WHERE m_map is NULL """
test {
sql """ DELETE FROM ${tableName} WHERE m_map is NULL """
exception("java.sql.SQLException: errCode = 2, detailMessage = errCode = 2, detailMessage = Can not apply delete condition to column type: MAP<INT,VARCHAR(30)>")
}
sql """ DELETE FROM ${tableName} WHERE id = 1; """
qt_sql """ SELECT * FROM ${tableName} order by id """
}

View File

@ -21,6 +21,10 @@ suite("test_struct_column_delete") {
sql """ DROP TABLE IF EXISTS ${tableName}; """
sql """ CREATE TABLE IF NOT EXISTS ${tableName} (id INT NULL, s_struct STRUCT<f1:INT, f2:VARCHAR(30)> NULL) ENGINE=OLAP DUPLICATE KEY(id) DISTRIBUTED BY HASH(id) BUCKETS 4 PROPERTIES ( "replication_allocation" = "tag.location.default: 1","in_memory" = "false","storage_format" = "V2") """
sql """ insert into ${tableName} values(1, {1, 'a'}),(2,NULL),(3,NULL),(4,NULL),(5,NULL) """
sql """ DELETE FROM ${tableName} WHERE s_struct is NULL """
test {
sql """ DELETE FROM ${tableName} WHERE s_struct is NULL """
exception("java.sql.SQLException: errCode = 2, detailMessage = errCode = 2, detailMessage = Can not apply delete condition to column type: STRUCT<f1:INT,f2:VARCHAR(30)>")
}
sql """ DELETE FROM ${tableName} WHERE id = 1; """
qt_sql """ SELECT * FROM ${tableName} order by id """
}

View File

@ -1,141 +0,0 @@
// Licensed to the Apache Software Foundation (ASF) under one
// or more contributor license agreements. See the NOTICE file
// distributed with this work for additional information
// regarding copyright ownership. The ASF licenses this file
// to you under the Apache License, Version 2.0 (the
// "License"); you may not use this file except in compliance
// with the License. You may obtain a copy of the License at
//
// http://www.apache.org/licenses/LICENSE-2.0
//
// Unless required by applicable law or agreed to in writing,
// software distributed under the License is distributed on an
// "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
// KIND, either express or implied. See the License for the
// specific language governing permissions and limitations
// under the License.
import org.codehaus.groovy.runtime.IOGroovyMethods
suite("test_index_change_7") {
def timeout = 60000
def delta_time = 1000
def alter_res = "null"
def useTime = 0
def wait_for_latest_op_on_table_finish = { table_name, OpTimeout ->
for(int t = delta_time; t <= OpTimeout; t += delta_time){
alter_res = sql """SHOW ALTER TABLE COLUMN WHERE TableName = "${table_name}" ORDER BY CreateTime DESC LIMIT 1;"""
alter_res = alter_res.toString()
if(alter_res.contains("FINISHED")) {
sleep(3000) // wait change table state to normal
logger.info(table_name + " latest alter job finished, detail: " + alter_res)
break
}
useTime = t
sleep(delta_time)
}
assertTrue(useTime <= OpTimeout, "wait_for_latest_op_on_table_finish timeout")
}
def wait_for_build_index_on_partition_finish = { table_name, OpTimeout ->
for(int t = delta_time; t <= OpTimeout; t += delta_time){
alter_res = sql """SHOW BUILD INDEX WHERE TableName = "${table_name}";"""
def expected_finished_num = alter_res.size();
def finished_num = 0;
for (int i = 0; i < expected_finished_num; i++) {
logger.info(table_name + " build index job state: " + alter_res[i][7] + i)
if (alter_res[i][7] == "FINISHED") {
++finished_num;
}
}
if (finished_num == expected_finished_num) {
logger.info(table_name + " all build index jobs finished, detail: " + alter_res)
break
}
useTime = t
sleep(delta_time)
}
assertTrue(useTime <= OpTimeout, "wait_for_latest_build_index_on_partition_finish timeout")
}
def tableName = "test_index_change_7"
sql """ DROP TABLE IF EXISTS ${tableName} """
sql """
CREATE TABLE IF NOT EXISTS ${tableName}(
`id`int(11)NULL,
`int_array` array<int(20)> NULL,
`c_array` array<varchar(20)> NULL,
INDEX c_array_idx(`c_array`) USING INVERTED PROPERTIES("parser"="english") COMMENT 'c_array index',
INDEX int_array_idx(`int_array`) USING INVERTED COMMENT 'int_array index'
) ENGINE=OLAP
DUPLICATE KEY(`id`)
COMMENT 'OLAP'
DISTRIBUTED BY HASH(`id`) BUCKETS 1
PROPERTIES(
"replication_allocation" = "tag.location.default: 1"
);
"""
sql "INSERT INTO ${tableName} VALUES (1, [10,20,30], ['i','love','china']), (2, [20,30,40], ['i','love','north korea']), (3, [30,40,50], NULL);"
sql "INSERT INTO ${tableName} VALUES (4, [40,50,60], NULL);"
qt_select1 """ SELECT * FROM ${tableName} t ORDER BY id; """
qt_select2 """ SELECT * FROM ${tableName} t WHERE c_array MATCH 'china' ORDER BY id; """
qt_select3 """ SELECT * FROM ${tableName} t WHERE c_array MATCH 'love' ORDER BY id; """
qt_select4 """ SELECT * FROM ${tableName} t WHERE c_array MATCH 'north' ORDER BY id; """
qt_select5 """ SELECT * FROM ${tableName} t WHERE c_array MATCH 'korea' ORDER BY id; """
qt_select6 """ SELECT * FROM ${tableName} t WHERE int_array element_ge 40 ORDER BY id; """
qt_select7 """ SELECT * FROM ${tableName} t WHERE int_array element_le 40 ORDER BY id; """
qt_select8 """ SELECT * FROM ${tableName} t WHERE int_array element_gt 40 ORDER BY id; """
qt_select9 """ SELECT * FROM ${tableName} t WHERE int_array element_lt 40 ORDER BY id; """
qt_select10 """ SELECT * FROM ${tableName} t WHERE int_array element_eq 10 ORDER BY id; """
qt_select11 """ SELECT * FROM ${tableName} t WHERE int_array element_eq 20 ORDER BY id; """
qt_select12 """ SELECT * FROM ${tableName} t WHERE int_array element_eq 30 ORDER BY id; """
qt_select13 """ SELECT * FROM ${tableName} t WHERE int_array element_eq 40 ORDER BY id; """
qt_select14 """ SELECT * FROM ${tableName} t WHERE int_array element_eq 50 ORDER BY id; """
qt_select15 """ SELECT * FROM ${tableName} t WHERE int_array element_eq 60 ORDER BY id; """
// drop inverted index c_array_idx, int_array_idx
sql """ DROP INDEX c_array_idx ON ${tableName} """
sql """ DROP INDEX int_array_idx ON ${tableName} """
// create inverted index
sql """ CREATE INDEX c_array_idx ON ${tableName}(`c_array`) USING INVERTED PROPERTIES("parser"="english") """
sql """ CREATE INDEX int_array_idx ON ${tableName}(`int_array`) USING INVERTED """
wait_for_latest_op_on_table_finish(tableName, timeout)
// build inverted index
sql """ BUILD INDEX c_array_idx ON ${tableName} """
sql """ BUILD INDEX int_array_idx ON ${tableName} """
wait_for_build_index_on_partition_finish(tableName, timeout)
def show_result = sql "show index from ${tableName}"
logger.info("show index from " + tableName + " result: " + show_result)
assertEquals(show_result.size(), 2)
assertEquals(show_result[0][2], "c_array_idx")
assertEquals(show_result[1][2], "int_array_idx")
qt_select16 """ SELECT * FROM ${tableName} t WHERE c_array MATCH 'china' ORDER BY id; """
qt_select17 """ SELECT * FROM ${tableName} t WHERE c_array MATCH 'love' ORDER BY id; """
qt_select18 """ SELECT * FROM ${tableName} t WHERE c_array MATCH 'north' ORDER BY id; """
qt_select19 """ SELECT * FROM ${tableName} t WHERE c_array MATCH 'korea' ORDER BY id; """
try {
qt_select20 """ SELECT * FROM ${tableName} t WHERE int_array element_ge 40 ORDER BY id; """
qt_select21 """ SELECT * FROM ${tableName} t WHERE int_array element_le 40 ORDER BY id; """
qt_select22 """ SELECT * FROM ${tableName} t WHERE int_array element_gt 40 ORDER BY id; """
qt_select23 """ SELECT * FROM ${tableName} t WHERE int_array element_lt 40 ORDER BY id; """
qt_select24 """ SELECT * FROM ${tableName} t WHERE int_array element_eq 10 ORDER BY id; """
qt_select25 """ SELECT * FROM ${tableName} t WHERE int_array element_eq 20 ORDER BY id; """
qt_select26 """ SELECT * FROM ${tableName} t WHERE int_array element_eq 30 ORDER BY id; """
qt_select27 """ SELECT * FROM ${tableName} t WHERE int_array element_eq 40 ORDER BY id; """
qt_select28 """ SELECT * FROM ${tableName} t WHERE int_array element_eq 50 ORDER BY id; """
qt_select29 """ SELECT * FROM ${tableName} t WHERE int_array element_eq 60 ORDER BY id; """
} catch(Exception ex) {
logger.info("execute array element query failed when build index not finished, result: " + ex)
}
}

View File

@ -1,88 +0,0 @@
// Licensed to the Apache Software Foundation (ASF) under one
// or more contributor license agreements. See the NOTICE file
// distributed with this work for additional information
// regarding copyright ownership. The ASF licenses this file
// to you under the Apache License, Version 2.0 (the
// "License"); you may not use this file except in compliance
// with the License. You may obtain a copy of the License at
//
// http://www.apache.org/licenses/LICENSE-2.0
//
// Unless required by applicable law or agreed to in writing,
// software distributed under the License is distributed on an
// "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
// KIND, either express or implied. See the License for the
// specific language governing permissions and limitations
// under the License.
suite("test_array_index"){
// prepare test table
def timeout = 60000
def delta_time = 1000
def alter_res = "null"
def useTime = 0
def indexTblName = "array_test"
sql "DROP TABLE IF EXISTS ${indexTblName}"
// create 1 replica table
sql """
CREATE TABLE IF NOT EXISTS ${indexTblName}(
`id`int(11)NULL,
`int_array` array<int(20)> NULL,
`c_array` array<varchar(20)> NULL,
INDEX c_array_idx(`c_array`) USING INVERTED PROPERTIES("parser"="english") COMMENT 'c_array index',
INDEX int_array_idx(`int_array`) USING INVERTED COMMENT 'int_array index'
) ENGINE=OLAP
DUPLICATE KEY(`id`)
COMMENT 'OLAP'
DISTRIBUTED BY HASH(`id`) BUCKETS 1
PROPERTIES(
"replication_allocation" = "tag.location.default: 1"
);
"""
def var_result = sql "show variables"
logger.info("show variales result: " + var_result )
sql "INSERT INTO $indexTblName VALUES (1, [10,20,30], ['i','love','china']), (2, [20,30,40], ['i','love','north korea']), (3, [30,40,50], NULL);"
sql "INSERT INTO $indexTblName VALUES (4, [40,50,60], NULL);"
qt_sql "SELECT * FROM $indexTblName WHERE c_array MATCH 'china' ORDER BY id;"
qt_sql "SELECT * FROM $indexTblName WHERE c_array MATCH 'love' ORDER BY id;"
qt_sql "SELECT * FROM $indexTblName WHERE c_array MATCH 'north' ORDER BY id;"
qt_sql "SELECT * FROM $indexTblName WHERE c_array MATCH 'korea' ORDER BY id;"
qt_sql "SELECT * FROM $indexTblName WHERE int_array element_ge 40 ORDER BY id;"
qt_sql "SELECT * FROM $indexTblName WHERE int_array element_le 40 ORDER BY id;"
qt_sql "SELECT * FROM $indexTblName WHERE int_array element_gt 40 ORDER BY id;"
qt_sql "SELECT * FROM $indexTblName WHERE int_array element_lt 40 ORDER BY id;"
qt_sql "SELECT * FROM $indexTblName WHERE int_array element_eq 10 ORDER BY id;"
qt_sql "SELECT * FROM $indexTblName WHERE int_array element_eq 20 ORDER BY id;"
qt_sql "SELECT * FROM $indexTblName WHERE int_array element_eq 30 ORDER BY id;"
qt_sql "SELECT * FROM $indexTblName WHERE int_array element_eq 40 ORDER BY id;"
qt_sql "SELECT * FROM $indexTblName WHERE int_array element_eq 50 ORDER BY id;"
qt_sql "SELECT * FROM $indexTblName WHERE int_array element_eq 60 ORDER BY id;"
sql "set enable_fallback_to_original_planner=false"
qt_sql "SELECT * FROM $indexTblName WHERE c_array MATCH 'china' ORDER BY id;"
qt_sql "SELECT * FROM $indexTblName WHERE c_array MATCH 'love' ORDER BY id;"
qt_sql "SELECT * FROM $indexTblName WHERE c_array MATCH 'north' ORDER BY id;"
qt_sql "SELECT * FROM $indexTblName WHERE c_array MATCH 'korea' ORDER BY id;"
sql "set enable_fallback_to_original_planner=true"
sql " ALTER TABLE $indexTblName drop index c_array_idx; "
qt_sql "SELECT * FROM $indexTblName WHERE c_array MATCH 'china' ORDER BY id;"
qt_sql "SELECT * FROM $indexTblName WHERE c_array MATCH 'love' ORDER BY id;"
qt_sql "SELECT * FROM $indexTblName WHERE c_array MATCH 'north' ORDER BY id;"
qt_sql "SELECT * FROM $indexTblName WHERE c_array MATCH 'korea' ORDER BY id;"
sql "set enable_fallback_to_original_planner=false"
qt_sql "SELECT * FROM $indexTblName WHERE c_array MATCH 'china' ORDER BY id;"
qt_sql "SELECT * FROM $indexTblName WHERE c_array MATCH 'love' ORDER BY id;"
qt_sql "SELECT * FROM $indexTblName WHERE c_array MATCH 'north' ORDER BY id;"
qt_sql "SELECT * FROM $indexTblName WHERE c_array MATCH 'korea' ORDER BY id;"
sql "set enable_fallback_to_original_planner=true"
}

View File

@ -0,0 +1,62 @@
// Licensed to the Apache Software Foundation (ASF) under one
// or more contributor license agreements. See the NOTICE file
// distributed with this work for additional information
// regarding copyright ownership. The ASF licenses this file
// to you under the Apache License, Version 2.0 (the
// "License"); you may not use this file except in compliance
// with the License. You may obtain a copy of the License at
//
// http://www.apache.org/licenses/LICENSE-2.0
//
// Unless required by applicable law or agreed to in writing,
// software distributed under the License is distributed on an
// "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
// KIND, either express or implied. See the License for the
// specific language governing permissions and limitations
// under the License.
suite("test_array_with_single_row_max_than_uint16", "query") {
// define a sql table
sql """set enable_nereids_planner=false"""
List<List<Object>> backends = sql """ select * from backends(); """
assertTrue(backends.size() > 0)
def be_id = backends[0][0]
def dataFilePath = context.config.dataPath + "/query_p0/"
def testTable = "arr_dd"
sql "DROP TABLE IF EXISTS ${testTable}"
def outFilePath= dataFilePath
if (backends.size() > 1) {
// cluster mode need to make sure all be has this data
outFilePath="/"
def transFile01 = "${dataFilePath}/arr_max.orc"
for (List<Object> backend : backends) {
def be_host = backend[1]
scpFiles("root", be_host, transFile01, outFilePath, false)
}
}
sql """
CREATE TABLE IF NOT EXISTS ${testTable} (
`k1` INT(11) NULL,
`k2` int(11) NULL,
`c_decimal` array<decimalv3(18, 5)> NULL DEFAULT "[]"
) ENGINE=OLAP
DUPLICATE KEY(`k1`)
COMMENT 'OLAP'
DISTRIBUTED BY HASH(`k1`) BUCKETS 1
PROPERTIES (
"replication_allocation" = "tag.location.default: 1",
"in_memory" = "false",
"storage_format" = "V2",
"disable_auto_compaction" = "false"
)
"""
// prepare data
qt_sql """
insert into ${testTable} select * from local(
"file_path" = "${outFilePath}/arr_max.orc",
"backend_id" = "${be_id}",
"format" = "orc");"""
qt_select """ select * from ${testTable} order by k1"""
}