Skip to content

Commit

Permalink
Merge branch 'master' into 20240827_load_trakcer_to_0
Browse files Browse the repository at this point in the history
  • Loading branch information
xinyiZzz authored Sep 2, 2024
2 parents e4b1e8d + 895dddf commit 00f4694
Show file tree
Hide file tree
Showing 6 changed files with 169 additions and 29 deletions.
6 changes: 4 additions & 2 deletions be/src/vec/common/sort/sorter.cpp
Original file line number Diff line number Diff line change
Expand Up @@ -93,8 +93,10 @@ Status MergeSorterState::merge_sort_read(doris::vectorized::Block* block, int ba
if (priority_queue_.empty()) {
*eos = true;
} else if (priority_queue_.size() == 1) {
if (offset_ != 0) {
priority_queue_.top().impl->block->skip_num_rows(offset_);
if (offset_ != 0 || priority_queue_.top()->pos != 0) {
// Skip rows already returned or need to be ignored
int64_t offset = offset_ + (int64_t)priority_queue_.top()->pos;
priority_queue_.top().impl->block->skip_num_rows(offset);
}
block->swap(*priority_queue_.top().impl->block);
*eos = true;
Expand Down
38 changes: 19 additions & 19 deletions be/test/vec/function/function_jsonb_test.cpp
Original file line number Diff line number Diff line change
Expand Up @@ -1239,12 +1239,12 @@ TEST(FunctionJsonbTEST, JsonbCastToOtherTest) {
{{STRING("null"), static_cast<uint8_t>(TypeIndex::UInt8)}, Null()},
{{STRING("true"), static_cast<uint8_t>(TypeIndex::UInt8)}, BOOLEAN(1)},
{{STRING("false"), static_cast<uint8_t>(TypeIndex::UInt8)}, BOOLEAN(0)},
{{STRING("100"), static_cast<uint8_t>(TypeIndex::UInt8)}, Null()}, //int8
{{STRING("10000"), static_cast<uint8_t>(TypeIndex::UInt8)}, Null()}, // int16
{{STRING("1000000000"), static_cast<uint8_t>(TypeIndex::UInt8)}, Null()}, // int32
{{STRING("100"), static_cast<uint8_t>(TypeIndex::UInt8)}, BOOLEAN(1)}, //int8
{{STRING("10000"), static_cast<uint8_t>(TypeIndex::UInt8)}, BOOLEAN(1)}, // int16
{{STRING("1000000000"), static_cast<uint8_t>(TypeIndex::UInt8)}, BOOLEAN(1)}, // int32
{{STRING("1152921504606846976"), static_cast<uint8_t>(TypeIndex::UInt8)},
Null()}, // int64
{{STRING("6.18"), static_cast<uint8_t>(TypeIndex::UInt8)}, Null()}, // double
BOOLEAN(1)}, // int64
{{STRING("6.18"), static_cast<uint8_t>(TypeIndex::UInt8)}, BOOLEAN(1)}, // double
{{STRING(R"("abcd")"), static_cast<uint8_t>(TypeIndex::UInt8)}, Null()}, // string
{{STRING("{}"), static_cast<uint8_t>(TypeIndex::UInt8)}, Null()}, // empty object
{{STRING(R"({"k1":"v31", "k2": 300})"), static_cast<uint8_t>(TypeIndex::UInt8)},
Expand All @@ -1269,14 +1269,14 @@ TEST(FunctionJsonbTEST, JsonbCastToOtherTest) {
// cast to TINYINT
data_set = {
{{STRING("null"), static_cast<int8_t>(TypeIndex::Int8)}, Null()},
{{STRING("true"), static_cast<int8_t>(TypeIndex::Int8)}, Null()},
{{STRING("false"), static_cast<int8_t>(TypeIndex::Int8)}, Null()},
{{STRING("true"), static_cast<int8_t>(TypeIndex::Int8)}, TINYINT(1)},
{{STRING("false"), static_cast<int8_t>(TypeIndex::Int8)}, TINYINT(0)},
{{STRING("100"), static_cast<int8_t>(TypeIndex::Int8)}, TINYINT(100)}, //int8
{{STRING("10000"), static_cast<int8_t>(TypeIndex::Int8)}, TINYINT(16)}, // int16
{{STRING("1000000000"), static_cast<int8_t>(TypeIndex::Int8)}, TINYINT(0)}, // int32
{{STRING("1152921504606846976"), static_cast<int8_t>(TypeIndex::Int8)},
TINYINT(0)}, // int64
{{STRING("6.18"), static_cast<int8_t>(TypeIndex::Int8)}, Null()}, // double
{{STRING("6.18"), static_cast<int8_t>(TypeIndex::Int8)}, TINYINT(6)}, // double
{{STRING(R"("abcd")"), static_cast<int8_t>(TypeIndex::Int8)}, Null()}, // string
{{STRING("{}"), static_cast<int8_t>(TypeIndex::Int8)}, Null()}, // empty object
{{STRING(R"({"k1":"v31", "k2": 300})"), static_cast<int8_t>(TypeIndex::Int8)},
Expand All @@ -1302,15 +1302,15 @@ TEST(FunctionJsonbTEST, JsonbCastToOtherTest) {
// cast to SMALLINT
data_set = {
{{STRING("null"), static_cast<int16_t>(TypeIndex::Int16)}, Null()},
{{STRING("true"), static_cast<int16_t>(TypeIndex::Int16)}, Null()},
{{STRING("false"), static_cast<int16_t>(TypeIndex::Int16)}, Null()},
{{STRING("true"), static_cast<int16_t>(TypeIndex::Int16)}, SMALLINT(1)},
{{STRING("false"), static_cast<int16_t>(TypeIndex::Int16)}, SMALLINT(0)},
{{STRING("100"), static_cast<int16_t>(TypeIndex::Int16)}, SMALLINT(100)}, //int8
{{STRING("10000"), static_cast<int16_t>(TypeIndex::Int16)}, SMALLINT(10000)}, // int16
{{STRING("1000000000"), static_cast<int16_t>(TypeIndex::Int16)},
SMALLINT(-13824)}, // int32
{{STRING("1152921504606846976"), static_cast<int16_t>(TypeIndex::Int16)},
SMALLINT(0)}, // int64
{{STRING("6.18"), static_cast<int16_t>(TypeIndex::Int16)}, Null()}, // double
{{STRING("6.18"), static_cast<int16_t>(TypeIndex::Int16)}, SMALLINT(6)}, // double
{{STRING(R"("abcd")"), static_cast<int16_t>(TypeIndex::Int16)}, Null()}, // string
{{STRING("{}"), static_cast<int16_t>(TypeIndex::Int16)}, Null()}, // empty object
{{STRING(R"({"k1":"v31", "k2": 300})"), static_cast<int16_t>(TypeIndex::Int16)},
Expand All @@ -1336,15 +1336,15 @@ TEST(FunctionJsonbTEST, JsonbCastToOtherTest) {
// cast to INT
data_set = {
{{STRING("null"), static_cast<int32_t>(TypeIndex::Int32)}, Null()},
{{STRING("true"), static_cast<int32_t>(TypeIndex::Int32)}, Null()},
{{STRING("false"), static_cast<int32_t>(TypeIndex::Int32)}, Null()},
{{STRING("true"), static_cast<int32_t>(TypeIndex::Int32)}, INT(1)},
{{STRING("false"), static_cast<int32_t>(TypeIndex::Int32)}, INT(0)},
{{STRING("100"), static_cast<int32_t>(TypeIndex::Int32)}, INT(100)}, //int8
{{STRING("10000"), static_cast<int32_t>(TypeIndex::Int32)}, INT(10000)}, // int16
{{STRING("1000000000"), static_cast<int32_t>(TypeIndex::Int32)},
INT(1000000000)}, // int32
{{STRING("1152921504606846976"), static_cast<int32_t>(TypeIndex::Int32)},
INT(0)}, // int64
{{STRING("6.18"), static_cast<int32_t>(TypeIndex::Int32)}, Null()}, // double
{{STRING("6.18"), static_cast<int32_t>(TypeIndex::Int32)}, INT(6)}, // double
{{STRING(R"("abcd")"), static_cast<int32_t>(TypeIndex::Int32)}, Null()}, // string
{{STRING("{}"), static_cast<int32_t>(TypeIndex::Int32)}, Null()}, // empty object
{{STRING(R"({"k1":"v31", "k2": 300})"), static_cast<int32_t>(TypeIndex::Int32)},
Expand All @@ -1370,13 +1370,13 @@ TEST(FunctionJsonbTEST, JsonbCastToOtherTest) {
// cast to BIGINT
data_set = {
{{STRING("null"), BIGINT(1)}, Null()},
{{STRING("true"), BIGINT(1)}, Null()},
{{STRING("false"), BIGINT(1)}, Null()},
{{STRING("true"), BIGINT(1)}, BIGINT(1)},
{{STRING("false"), BIGINT(1)}, BIGINT(0)},
{{STRING("100"), BIGINT(1)}, BIGINT(100)}, //int8
{{STRING("10000"), BIGINT(1)}, BIGINT(10000)}, // int16
{{STRING("1000000000"), BIGINT(1)}, BIGINT(1000000000)}, // int32
{{STRING("1152921504606846976"), BIGINT(1)}, BIGINT(1152921504606846976)}, // int64
{{STRING("6.18"), BIGINT(1)}, Null()}, // double
{{STRING("6.18"), BIGINT(1)}, BIGINT(6)}, // double
{{STRING(R"("abcd")"), BIGINT(1)}, Null()}, // string
{{STRING("{}"), BIGINT(1)}, Null()}, // empty object
{{STRING(R"({"k1":"v31", "k2": 300})"), BIGINT(1)}, Null()}, // object
Expand All @@ -1398,8 +1398,8 @@ TEST(FunctionJsonbTEST, JsonbCastToOtherTest) {
// cast to DOUBLE
data_set = {
{{STRING("null"), DOUBLE(1)}, Null()},
{{STRING("true"), DOUBLE(1)}, Null()},
{{STRING("false"), DOUBLE(1)}, Null()},
{{STRING("true"), DOUBLE(1)}, DOUBLE(1)},
{{STRING("false"), DOUBLE(1)}, DOUBLE(0)},
{{STRING("100"), DOUBLE(1)}, DOUBLE(100)}, //int8
{{STRING("10000"), DOUBLE(1)}, DOUBLE(10000)}, // int16
{{STRING("1000000000"), DOUBLE(1)}, DOUBLE(1000000000)}, // int32
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -3350,7 +3350,7 @@ public TCreatePartitionResult createPartition(TCreatePartitionRequest request) t
return result;
}

Table table = db.getTable(tableId).get();
OlapTable table = (OlapTable) (db.getTable(tableId).get());
if (table == null) {
errorStatus.setErrorMsgs(
(Lists.newArrayList(String.format("dbId=%d tableId=%d is not exists", dbId, tableId))));
Expand Down Expand Up @@ -3432,6 +3432,16 @@ public TCreatePartitionResult createPartition(TCreatePartitionRequest request) t
List<TTabletLocation> tablets = Lists.newArrayList();
for (String partitionName : addPartitionClauseMap.keySet()) {
Partition partition = table.getPartition(partitionName);
if (partition == null) {
String partInfos = table.getAllPartitions().stream()
.map(partitionArg -> partitionArg.getName() + partitionArg.getId())
.collect(Collectors.joining(", "));

errorStatus.setErrorMsgs(Lists.newArrayList("get partition " + partitionName + " failed"));
result.setStatus(errorStatus);
LOG.warn("send create partition error status: {}, {}", result, partInfos);
return result;
}
TOlapTablePartition tPartition = new TOlapTablePartition();
tPartition.setId(partition.getId());
int partColNum = partitionInfo.getPartitionColumns().size();
Expand Down
12 changes: 12 additions & 0 deletions regression-test/data/query_p0/operator/test_sort_operator.out
Original file line number Diff line number Diff line change
@@ -0,0 +1,12 @@
-- This file is automatically generated. You should know what you did if you want to edit this
-- !select --
100174 \N
100271 \N
100271 \N
100271 \N
100471 \N
100471 \N
100471 \N
100567 \N
100567 \N

Original file line number Diff line number Diff line change
Expand Up @@ -210,12 +210,13 @@ suite("test_single_compaction_with_variant_inverted", "p2") {
}
}

sql """ INSERT INTO ${tableName} VALUES (1, "a", 100, '{"a" : 1234, "point" : 1, "xxxx" : "ddddd"}'); """
sql """ INSERT INTO ${tableName} VALUES (1, "b", 100, '{"%a" : 1234, "@point" : 1, "[xxxx" : "ddddd"}'); """
sql """ INSERT INTO ${tableName} VALUES (2, "a", 100, '{"@a" : 1234, "%point" : 1, "]xxxx" : "ddddd"}'); """
sql """ INSERT INTO ${tableName} VALUES (2, "b", 100, '{"%a" : 1234, "%point" : 1, "{xxxx" : "ddddd"}'); """
sql """ INSERT INTO ${tableName} VALUES (3, "a", 100, '{"@a" : 1234, "@point" : 1, "}xxxx" : "ddddd"}'); """
sql """ INSERT INTO ${tableName} VALUES (3, "b", 100, '{"a" : 1234, "point" : 1, "|xxxx" : "ddddd"}'); """
sql """ INSERT INTO ${tableName} VALUES (1, "b", 100, '{"%a" : 1234, "@point" : 1, "xxxx" : "ddddd"}'); """
sql """ INSERT INTO ${tableName} VALUES (2, "a", 100, '{"@a" : 1234, "%point" : 1, "xxxx" : "ddddd"}'); """
sql """ INSERT INTO ${tableName} VALUES (2, "b", 100, '{"%a" : 1234, "%point" : 1, "xxxx" : "ddddd"}'); """
sql """ INSERT INTO ${tableName} VALUES (3, "a", 100, '{"@a" : 1234, "@point" : 1, "xxxx" : "ddddd"}'); """
sql """ INSERT INTO ${tableName} VALUES (3, "b", 100, '{"a" : 1234, "point" : 1, "xxxx" : "ddddd"}'); """
sql """ INSERT INTO ${tableName} VALUES (3, "b", 100, '{"a" : 1234, "point" : 1, "xxxx" : "ddddd"}'); """


// trigger master be to do full compaction
assertTrue(triggerCompaction(backendId_to_backendIP[master_backend_id], backendId_to_backendHttpPort[master_backend_id],
Expand All @@ -233,7 +234,7 @@ suite("test_single_compaction_with_variant_inverted", "p2") {
checkTabletFileCrc.call()

qt_sql """
select count() from ${tableName} where properties MATCH_ANY 'point xxxx';
select count() from ${tableName} where cast(properties['xxxx'] as string) MATCH_ANY 'ddddd';
"""

sql """ DROP TABLE IF EXISTS ${tableName}; """
Expand Down
115 changes: 115 additions & 0 deletions regression-test/suites/query_p0/operator/test_sort_operator.groovy
Original file line number Diff line number Diff line change
@@ -0,0 +1,115 @@
// 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_sort_operator", "query,p0,arrow_flight_sql") {

sql """
DROP TABLE IF EXISTS dim_org_ful;
"""

sql """
CREATE TABLE `dim_org_ful` (
`org_id` int(11) NOT NULL COMMENT '',
`start_dt` date NOT NULL COMMENT '',
`end_dt` date REPLACE_IF_NOT_NULL NULL COMMENT ''
) ENGINE=OLAP
AGGREGATE KEY(`org_id`, `start_dt`)
COMMENT '网点'
DISTRIBUTED BY HASH(`org_id`) BUCKETS 3
PROPERTIES (
"replication_allocation" = "tag.location.default: 1",
"is_being_synced" = "false",
"storage_format" = "V2",
"light_schema_change" = "true",
"disable_auto_compaction" = "false",
"enable_single_replica_compaction" = "false"
);
"""

sql """
DROP TABLE IF EXISTS dim_day;
"""

sql """
CREATE TABLE `dim_day` (
`day_key` varchar(80) NULL,
`day_date` date NULL
) ENGINE=OLAP
DUPLICATE KEY(`day_key`)
COMMENT 'OLAP'
DISTRIBUTED BY HASH(`day_key`, `day_date`) BUCKETS 10
PROPERTIES (
"replication_allocation" = "tag.location.default: 1",
"is_being_synced" = "false",
"storage_format" = "V2",
"light_schema_change" = "true",
"disable_auto_compaction" = "false"
);
"""

sql """
INSERT INTO `dim_day` VALUES
('20231006','2023-10-06'),
('20231010','2023-10-10'),
('20230822','2023-08-22'),
('20230829','2023-08-29'),
('20230925','2023-09-25'),
('20230731','2023-07-31'),
('20230928','2023-09-28'),
('20230727','2023-07-27'),
('20230801','2023-08-01'),
('20231017','2023-10-17');
"""

sql """INSERT INTO `dim_org_ful` VALUES
(20,'2023-08-02','3000-12-31'),
(100174,'2023-07-31','2023-08-01'),
(100174,'2023-08-01','3000-12-31'),
(100271,'2023-07-26','3000-12-31'),
(100424,'2023-08-02','3000-12-31'),
(100471,'2023-07-26','3000-12-31'),
(100567,'2023-07-29','2023-07-30'),
(100567,'2023-07-30','2023-07-31'),
(100567,'2023-07-31','3000-12-31'),
(100723,'2023-07-30','2023-07-31');"""

sql """
set batch_size = 9;
"""
sql """
set force_sort_algorithm="topn";
"""
sql """
set parallel_pipeline_task_num=1;
"""

order_qt_select """
with `dim_org` AS(
SELECT
`t0`.`day_date` AS `ds`,
`org_id` AS `org_id`
FROM
`dim_day` t0
INNER JOIN `dim_org_ful` t1 ON `t0`.`day_date` BETWEEN `t1`.`start_dt`
AND `t1`.`end_dt` - 1.0
WHERE
`t0`.`day_date` BETWEEN '2021-01-01 00:00:00'
AND '2023-08-07'
)
select org_id,null from dim_org order by 1,2 limit 1,10
"""
}

0 comments on commit 00f4694

Please sign in to comment.