This is an automated email from the ASF dual-hosted git repository.
dataroaring pushed a commit to branch branch-3.0
in repository https://gitbox.apache.org/repos/asf/doris.git
The following commit(s) were added to refs/heads/branch-3.0 by this push:
new 34d83429949 branch-3.0: [Enhancement] add information_schema
backend_tablets table (#52195)
34d83429949 is described below
commit 34d83429949b5b1866450ef3200a44b8a8cae18b
Author: koarz <[email protected]>
AuthorDate: Sat Jun 28 17:30:33 2025 +0800
branch-3.0: [Enhancement] add information_schema backend_tablets table
(#52195)
https://github.com/apache/doris/pull/51282
---
be/src/cloud/cloud_schema_change_job.cpp | 9 +-
be/src/cloud/cloud_tablet.cpp | 4 +
be/src/cloud/cloud_tablet.h | 2 +
be/src/cloud/cloud_tablet_mgr.cpp | 7 +
be/src/cloud/cloud_tablet_mgr.h | 2 +
be/src/exec/schema_scanner.cpp | 3 +
.../exec/schema_scanner/schema_scanner_helper.cpp | 43 +++-
be/src/exec/schema_scanner/schema_scanner_helper.h | 5 +-
.../exec/schema_scanner/schema_tablets_scanner.cpp | 227 +++++++++++++++++++++
...a_scanner_helper.h => schema_tablets_scanner.h} | 41 ++--
be/src/olap/base_tablet.h | 12 ++
be/src/olap/tablet.h | 17 +-
.../org/apache/doris/analysis/SchemaTableType.java | 1 +
.../java/org/apache/doris/catalog/SchemaTable.java | 23 ++-
.../planner/BackendPartitionedSchemaScanNode.java | 1 +
gensrc/thrift/Descriptors.thrift | 3 +-
.../info_schema_db/backend_tablets.out | Bin 0 -> 110 bytes
.../info_schema_db/backend_tablets.groovy | 126 ++++++++++++
18 files changed, 483 insertions(+), 43 deletions(-)
diff --git a/be/src/cloud/cloud_schema_change_job.cpp
b/be/src/cloud/cloud_schema_change_job.cpp
index 05d29383af4..a5707e51bb6 100644
--- a/be/src/cloud/cloud_schema_change_job.cpp
+++ b/be/src/cloud/cloud_schema_change_job.cpp
@@ -80,6 +80,13 @@ Status CloudSchemaChangeJob::process_alter_tablet(const
TAlterTabletReqV2& reque
std::unique_lock<std::mutex>
schema_change_lock(_base_tablet->get_schema_change_lock(),
std::try_to_lock);
+ _new_tablet->set_alter_failed(false);
+ Defer defer([this] {
+ // if tablet state is not TABLET_RUNNING when return, indicates that
alter has failed.
+ if (_new_tablet->tablet_state() != TABLET_RUNNING) {
+ _new_tablet->set_alter_failed(true);
+ }
+ });
if (!schema_change_lock.owns_lock()) {
LOG(WARNING) << "Failed to obtain schema change lock. base_tablet="
<< request.base_tablet_id;
@@ -134,7 +141,7 @@ Status CloudSchemaChangeJob::process_alter_tablet(const
TAlterTabletReqV2& reque
RETURN_IF_ERROR(_base_tablet->capture_rs_readers({2,
start_resp.alter_version()},
&rs_splits, false));
}
- Defer defer {[&]() {
+ Defer defer2 {[&]() {
_new_tablet->set_alter_version(-1);
_base_tablet->set_alter_version(-1);
}};
diff --git a/be/src/cloud/cloud_tablet.cpp b/be/src/cloud/cloud_tablet.cpp
index a26ce0dd2c7..582a4b77126 100644
--- a/be/src/cloud/cloud_tablet.cpp
+++ b/be/src/cloud/cloud_tablet.cpp
@@ -86,6 +86,10 @@ Status CloudTablet::capture_consistent_rowsets_unlocked(
return _capture_consistent_rowsets_unlocked(version_path, rowsets);
}
+std::string CloudTablet::tablet_path() const {
+ return "";
+}
+
Status CloudTablet::capture_rs_readers(const Version& spec_version,
std::vector<RowSetSplits>* rs_splits,
bool skip_missing_version) {
diff --git a/be/src/cloud/cloud_tablet.h b/be/src/cloud/cloud_tablet.h
index f3af8b09b27..2dd1d3c4425 100644
--- a/be/src/cloud/cloud_tablet.h
+++ b/be/src/cloud/cloud_tablet.h
@@ -62,6 +62,8 @@ public:
return _approximate_data_size.load(std::memory_order_relaxed);
}
+ std::string tablet_path() const override;
+
// clang-format off
int64_t fetch_add_approximate_num_rowsets (int64_t x) { return
_approximate_num_rowsets .fetch_add(x, std::memory_order_relaxed); }
int64_t fetch_add_approximate_num_segments(int64_t x) { return
_approximate_num_segments.fetch_add(x, std::memory_order_relaxed); }
diff --git a/be/src/cloud/cloud_tablet_mgr.cpp
b/be/src/cloud/cloud_tablet_mgr.cpp
index efcfdde7361..cbf9a29ee90 100644
--- a/be/src/cloud/cloud_tablet_mgr.cpp
+++ b/be/src/cloud/cloud_tablet_mgr.cpp
@@ -508,6 +508,13 @@ void CloudTabletMgr::get_topn_tablet_delete_bitmap_score(
<< max_base_rowset_delete_bitmap_score_tablet_id << ",tablets=["
<< ss.str() << "]";
}
+std::vector<std::shared_ptr<CloudTablet>> CloudTabletMgr::get_all_tablet() {
+ std::vector<std::shared_ptr<CloudTablet>> tablets;
+ tablets.reserve(_tablet_map->size());
+ _tablet_map->traverse([&tablets](auto& t) { tablets.push_back(t); });
+ return tablets;
+}
+
void CloudTabletMgr::put_tablet_for_UT(std::shared_ptr<CloudTablet> tablet) {
_tablet_map->put(tablet);
}
diff --git a/be/src/cloud/cloud_tablet_mgr.h b/be/src/cloud/cloud_tablet_mgr.h
index c8ff133df82..a1ce6d2b8cf 100644
--- a/be/src/cloud/cloud_tablet_mgr.h
+++ b/be/src/cloud/cloud_tablet_mgr.h
@@ -91,6 +91,8 @@ public:
// **ATTN: JUST FOR UT**
void put_tablet_for_UT(std::shared_ptr<CloudTablet> tablet);
+ std::vector<std::shared_ptr<CloudTablet>> get_all_tablet();
+
private:
CloudStorageEngine& _engine;
diff --git a/be/src/exec/schema_scanner.cpp b/be/src/exec/schema_scanner.cpp
index e9d068baca6..5f60ffb7bc6 100644
--- a/be/src/exec/schema_scanner.cpp
+++ b/be/src/exec/schema_scanner.cpp
@@ -49,6 +49,7 @@
#include "exec/schema_scanner/schema_table_privileges_scanner.h"
#include "exec/schema_scanner/schema_table_properties_scanner.h"
#include "exec/schema_scanner/schema_tables_scanner.h"
+#include "exec/schema_scanner/schema_tablets_scanner.h"
#include "exec/schema_scanner/schema_user_privileges_scanner.h"
#include "exec/schema_scanner/schema_user_scanner.h"
#include "exec/schema_scanner/schema_variables_scanner.h"
@@ -247,6 +248,8 @@ std::unique_ptr<SchemaScanner>
SchemaScanner::create(TSchemaTableType::type type
return SchemaCatalogMetaCacheStatsScanner::create_unique();
case TSchemaTableType::SCH_ROUTINE_LOAD_JOBS:
return SchemaRoutineLoadJobScanner::create_unique();
+ case TSchemaTableType::SCH_BACKEND_TABLETS:
+ return SchemaTabletsScanner::create_unique();
default:
return SchemaDummyScanner::create_unique();
break;
diff --git a/be/src/exec/schema_scanner/schema_scanner_helper.cpp
b/be/src/exec/schema_scanner/schema_scanner_helper.cpp
index 0fea9d8c39f..5f4af06c52d 100644
--- a/be/src/exec/schema_scanner/schema_scanner_helper.cpp
+++ b/be/src/exec/schema_scanner/schema_scanner_helper.cpp
@@ -30,29 +30,54 @@ namespace doris {
void SchemaScannerHelper::insert_string_value(int col_index, std::string
str_val,
vectorized::Block* block) {
vectorized::MutableColumnPtr mutable_col_ptr;
- mutable_col_ptr =
std::move(*block->get_by_position(col_index).column).assume_mutable();
+ mutable_col_ptr =
block->get_by_position(col_index).column->assume_mutable();
auto* nullable_column =
assert_cast<vectorized::ColumnNullable*>(mutable_col_ptr.get());
vectorized::IColumn* col_ptr = &nullable_column->get_nested_column();
assert_cast<vectorized::ColumnString*>(col_ptr)->insert_data(str_val.data(),
str_val.size());
nullable_column->get_null_map_data().emplace_back(0);
}
-void SchemaScannerHelper::insert_datetime_value(int col_index, const
std::vector<void*>& datas,
- vectorized::Block* block) {
+void SchemaScannerHelper::insert_datetime_value(int col_index, int64_t
timestamp,
+ const std::string& ctz,
vectorized::Block* block) {
vectorized::MutableColumnPtr mutable_col_ptr;
- mutable_col_ptr =
std::move(*block->get_by_position(col_index).column).assume_mutable();
+ mutable_col_ptr =
block->get_by_position(col_index).column->assume_mutable();
auto* nullable_column =
assert_cast<vectorized::ColumnNullable*>(mutable_col_ptr.get());
vectorized::IColumn* col_ptr = &nullable_column->get_nested_column();
- auto data = datas[0];
-
assert_cast<vectorized::ColumnVector<vectorized::Int64>*>(col_ptr)->insert_data(
- reinterpret_cast<char*>(data), 0);
+
+ std::vector<void*> datas(1);
+ VecDateTimeValue src[1];
+ src[0].from_unixtime(timestamp, ctz);
+ datas[0] = src;
+ auto* data = datas[0];
+
assert_cast<vectorized::ColumnDateTime*>(col_ptr)->insert_data(reinterpret_cast<char*>(data),
+ 0);
+ nullable_column->get_null_map_data().emplace_back(0);
+}
+
+void SchemaScannerHelper::insert_bool_value(int col_index, bool bool_val,
+ vectorized::Block* block) {
+ vectorized::MutableColumnPtr mutable_col_ptr;
+ mutable_col_ptr =
block->get_by_position(col_index).column->assume_mutable();
+ auto* nullable_column =
assert_cast<vectorized::ColumnNullable*>(mutable_col_ptr.get());
+ vectorized::IColumn* col_ptr = &nullable_column->get_nested_column();
+ assert_cast<vectorized::ColumnBool*>(col_ptr)->insert_value(bool_val);
+ nullable_column->get_null_map_data().emplace_back(0);
+}
+
+void SchemaScannerHelper::insert_int32_value(int col_index, int32_t int_val,
+ vectorized::Block* block) {
+ vectorized::MutableColumnPtr mutable_col_ptr;
+ mutable_col_ptr =
block->get_by_position(col_index).column->assume_mutable();
+ auto* nullable_column =
assert_cast<vectorized::ColumnNullable*>(mutable_col_ptr.get());
+ vectorized::IColumn* col_ptr = &nullable_column->get_nested_column();
+ assert_cast<vectorized::ColumnInt32*>(col_ptr)->insert_value(int_val);
nullable_column->get_null_map_data().emplace_back(0);
}
void SchemaScannerHelper::insert_int64_value(int col_index, int64_t int_val,
vectorized::Block* block) {
vectorized::MutableColumnPtr mutable_col_ptr;
- mutable_col_ptr =
std::move(*block->get_by_position(col_index).column).assume_mutable();
+ mutable_col_ptr =
block->get_by_position(col_index).column->assume_mutable();
auto* nullable_column =
assert_cast<vectorized::ColumnNullable*>(mutable_col_ptr.get());
vectorized::IColumn* col_ptr = &nullable_column->get_nested_column();
assert_cast<vectorized::ColumnVector<vectorized::Int64>*>(col_ptr)->insert_value(int_val);
@@ -62,7 +87,7 @@ void SchemaScannerHelper::insert_int64_value(int col_index,
int64_t int_val,
void SchemaScannerHelper::insert_double_value(int col_index, double double_val,
vectorized::Block* block) {
vectorized::MutableColumnPtr mutable_col_ptr;
- mutable_col_ptr =
std::move(*block->get_by_position(col_index).column).assume_mutable();
+ mutable_col_ptr =
block->get_by_position(col_index).column->assume_mutable();
auto* nullable_column =
assert_cast<vectorized::ColumnNullable*>(mutable_col_ptr.get());
vectorized::IColumn* col_ptr = &nullable_column->get_nested_column();
assert_cast<vectorized::ColumnVector<vectorized::Float64>*>(col_ptr)->insert_value(double_val);
diff --git a/be/src/exec/schema_scanner/schema_scanner_helper.h
b/be/src/exec/schema_scanner/schema_scanner_helper.h
index f7b47ede91b..aed14677363 100644
--- a/be/src/exec/schema_scanner/schema_scanner_helper.h
+++ b/be/src/exec/schema_scanner/schema_scanner_helper.h
@@ -32,9 +32,12 @@ class Block;
class SchemaScannerHelper {
public:
static void insert_string_value(int col_index, std::string str_val,
vectorized::Block* block);
- static void insert_datetime_value(int col_index, const std::vector<void*>&
datas,
+ static void insert_datetime_value(int col_index, int64_t timestamp, const
std::string& ctz,
vectorized::Block* block);
+ static void insert_bool_value(int col_index, bool bool_val,
vectorized::Block* block);
+
+ static void insert_int32_value(int col_index, int32_t int_val,
vectorized::Block* block);
static void insert_int64_value(int col_index, int64_t int_val,
vectorized::Block* block);
static void insert_double_value(int col_index, double double_val,
vectorized::Block* block);
};
diff --git a/be/src/exec/schema_scanner/schema_tablets_scanner.cpp
b/be/src/exec/schema_scanner/schema_tablets_scanner.cpp
new file mode 100644
index 00000000000..972492e9151
--- /dev/null
+++ b/be/src/exec/schema_scanner/schema_tablets_scanner.cpp
@@ -0,0 +1,227 @@
+// 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.
+
+#include "exec/schema_scanner/schema_tablets_scanner.h"
+
+#include <gen_cpp/Descriptors_types.h>
+#include <gen_cpp/olap_common.pb.h>
+
+#include <algorithm>
+#include <cstddef>
+#include <cstdint>
+#include <memory>
+#include <numeric>
+#include <string>
+#include <utility>
+
+#include "cloud/cloud_storage_engine.h"
+#include "cloud/cloud_tablet.h"
+#include "cloud/cloud_tablet_mgr.h"
+#include "cloud/config.h"
+#include "common/status.h"
+#include "exec/schema_scanner.h"
+#include "exec/schema_scanner/schema_scanner_helper.h"
+#include "olap/storage_engine.h"
+#include "olap/tablet_fwd.h"
+#include "olap/tablet_manager.h"
+#include "runtime/define_primitive_type.h"
+#include "runtime/exec_env.h"
+#include "runtime/runtime_state.h"
+#include "vec/common/string_ref.h"
+
+namespace doris {
+namespace vectorized {
+class Block;
+} // namespace vectorized
+
+#include "common/compile_check_begin.h"
+
+std::vector<SchemaScanner::ColumnDesc> SchemaTabletsScanner::_s_tbls_columns =
{
+ // name, type, size, is_null
+ {"BE_ID", TYPE_BIGINT, sizeof(int64_t), true},
+ {"TABLET_ID", TYPE_BIGINT, sizeof(int64_t), true},
+ {"REPLICA_ID", TYPE_BIGINT, sizeof(int64_t), true},
+ {"PARTITION_ID", TYPE_BIGINT, sizeof(int64_t), true},
+ {"TABLET_PATH", TYPE_STRING, sizeof(StringRef), true},
+ {"TABLET_LOCAL_SIZE", TYPE_BIGINT, sizeof(int64_t), true},
+ {"TABLET_REMOTE_SIZE", TYPE_BIGINT, sizeof(int64_t), true},
+ {"VERSION_COUNT", TYPE_BIGINT, sizeof(int64_t), true},
+ {"SEGMENT_COUNT", TYPE_BIGINT, sizeof(int64_t), true},
+ {"NUM_COLUMNS", TYPE_BIGINT, sizeof(int64_t), true},
+ {"ROW_SIZE", TYPE_BIGINT, sizeof(int64_t), true},
+ {"COMPACTION_SCORE", TYPE_INT, sizeof(int32_t), true},
+ {"COMPRESS_KIND", TYPE_STRING, sizeof(StringRef), true},
+ {"IS_USED", TYPE_BOOLEAN, sizeof(bool), true},
+ {"IS_ALTER_FAILED", TYPE_BOOLEAN, sizeof(bool), true},
+ {"CREATE_TIME", TYPE_DATETIME, sizeof(int64_t), true},
+ {"UPDATE_TIME", TYPE_DATETIME, sizeof(int64_t), true},
+ {"IS_OVERLAP", TYPE_BOOLEAN, sizeof(bool), true},
+};
+
+SchemaTabletsScanner::SchemaTabletsScanner()
+ : SchemaScanner(_s_tbls_columns,
TSchemaTableType::SCH_BACKEND_TABLETS) {};
+
+Status SchemaTabletsScanner::start(RuntimeState* state) {
+ if (!_is_init) {
+ return Status::InternalError("used before initialized.");
+ }
+ _backend_id = state->backend_id();
+ RETURN_IF_ERROR(_get_all_tablets());
+ return Status::OK();
+}
+
+Status SchemaTabletsScanner::_get_all_tablets() {
+ if (config::is_cloud_mode()) {
+ auto tablets =
+
ExecEnv::GetInstance()->storage_engine().to_cloud().tablet_mgr().get_all_tablet();
+ std::ranges::for_each(tablets, [&](auto& tablet) {
+ _tablets.push_back(std::static_pointer_cast<BaseTablet>(tablet));
+ });
+ } else {
+ auto tablets = ExecEnv::GetInstance()
+ ->storage_engine()
+ .to_local()
+ .tablet_manager()
+ ->get_all_tablet();
+ std::ranges::for_each(tablets, [&](auto& tablet) {
+ _tablets.push_back(std::static_pointer_cast<BaseTablet>(tablet));
+ });
+ }
+ return Status::OK();
+}
+
+Status SchemaTabletsScanner::get_next_block_internal(vectorized::Block* block,
bool* eos) {
+ if (!_is_init) {
+ return Status::InternalError("Used before initialized.");
+ }
+ if (nullptr == block || nullptr == eos) {
+ return Status::InternalError("input pointer is nullptr.");
+ }
+ *eos = true;
+ return _fill_block_impl(block);
+}
+
+Status SchemaTabletsScanner::_fill_block_impl(vectorized::Block* block) {
+ SCOPED_TIMER(_fill_block_timer);
+
+ size_t row_num = _tablets.size();
+ if (row_num == 0) {
+ return Status::OK();
+ }
+
+ size_t fill_tablets_num = _tablets.size();
+ std::vector<void*> datas(fill_tablets_num);
+
+ for (int i = 0; i < _tablets.size(); i++) {
+ BaseTabletSPtr tablet = _tablets[i];
+ // BE_ID
+ SchemaScannerHelper::insert_int64_value(0, _backend_id, block);
+
+ // TABLET_ID
+ SchemaScannerHelper::insert_int64_value(1,
tablet->tablet_meta()->tablet_id(), block);
+
+ // REPLICA_ID
+ SchemaScannerHelper::insert_int64_value(2,
tablet->tablet_meta()->replica_id(), block);
+
+ // PARTITION_ID
+ SchemaScannerHelper::insert_int64_value(3,
tablet->tablet_meta()->partition_id(), block);
+
+ // TABLET_PATH
+ SchemaScannerHelper::insert_string_value(4, tablet->tablet_path(),
block);
+
+ // TABLET_LOCAL_SIZE
+ SchemaScannerHelper::insert_int64_value(5,
tablet->tablet_meta()->tablet_local_size(),
+ block);
+
+ // TABLET_REMOTE_SIZE
+ SchemaScannerHelper::insert_int64_value(6,
tablet->tablet_meta()->tablet_remote_size(),
+ block);
+
+ // VERSION_COUNT
+ SchemaScannerHelper::insert_int64_value(
+ 7,
static_cast<int64_t>(tablet->tablet_meta()->version_count()), block);
+
+ // SEGMENT_COUNT
+ SchemaScannerHelper::insert_int64_value(
+ 8,
+ [&tablet]() {
+ auto rs_metas = tablet->tablet_meta()->all_rs_metas();
+ return std::accumulate(rs_metas.begin(), rs_metas.end(), 0,
+ [](int64_t val,
RowsetMetaSharedPtr& rs_meta) {
+ return val +
rs_meta->num_segments();
+ });
+ }(),
+ block);
+
+ // NUM_COLUMNS
+ SchemaScannerHelper::insert_int64_value(9,
tablet->tablet_meta()->tablet_columns_num(),
+ block);
+
+ // ROW_SIZE
+ SchemaScannerHelper::insert_int64_value(10,
static_cast<int64_t>(tablet->row_size()),
+ block);
+
+ // COMPACTION_SCORE
+ SchemaScannerHelper::insert_int32_value(11,
tablet->get_real_compaction_score(), block);
+
+ // COMPRESS_KIND
+ SchemaScannerHelper::insert_string_value(12,
CompressKind_Name(tablet->compress_kind()),
+ block);
+
+ // IS_USED
+ SchemaScannerHelper::insert_bool_value(
+ 13,
+ [&tablet]() {
+ if (config::is_cloud_mode()) {
+ return true;
+ }
+ return std::static_pointer_cast<Tablet>(tablet)->is_used();
+ }(),
+ block);
+
+ // IS_ALTER_FAILED
+ SchemaScannerHelper::insert_bool_value(14, tablet->is_alter_failed(),
block);
+
+ // CREATE_TIME
+ SchemaScannerHelper::insert_datetime_value(15,
tablet->tablet_meta()->creation_time(),
+
TimezoneUtils::default_time_zone, block);
+
+ // UPDATE_TIME
+ SchemaScannerHelper::insert_datetime_value(
+ 16,
+ [&tablet]() {
+ auto rowset = tablet->get_rowset_with_max_version();
+ return rowset == nullptr ? 0 :
rowset->newest_write_timestamp();
+ }(),
+ TimezoneUtils::default_time_zone, block);
+
+ // IS_OVERLAP
+ SchemaScannerHelper::insert_bool_value(
+ 17,
+ [&tablet]() {
+ const auto& rs_metas =
tablet->tablet_meta()->all_rs_metas();
+ return std::any_of(rs_metas.begin(), rs_metas.end(),
+ [](const RowsetMetaSharedPtr& rs_meta) {
+ return
rs_meta->is_segments_overlapping();
+ });
+ }(),
+ block);
+ }
+
+ return Status::OK();
+}
+} // namespace doris
diff --git a/be/src/exec/schema_scanner/schema_scanner_helper.h
b/be/src/exec/schema_scanner/schema_tablets_scanner.h
similarity index 57%
copy from be/src/exec/schema_scanner/schema_scanner_helper.h
copy to be/src/exec/schema_scanner/schema_tablets_scanner.h
index f7b47ede91b..dc8b7706724 100644
--- a/be/src/exec/schema_scanner/schema_scanner_helper.h
+++ b/be/src/exec/schema_scanner/schema_tablets_scanner.h
@@ -15,29 +15,42 @@
// specific language governing permissions and limitations
// under the License.
-#ifndef _SCHEMA_SCANNER_HELPER_H_
+#pragma once
-#include <stdint.h>
-
-#include <string>
+#include <cstddef>
+#include <cstdint>
#include <vector>
-// this is a util class which can be used by all shema scanner
-// all common functions are added in this class.
+#include "common/status.h"
+#include "exec/schema_scanner.h"
+#include "olap/tablet.h"
+
namespace doris {
+class RuntimeState;
namespace vectorized {
class Block;
} // namespace vectorized
-class SchemaScannerHelper {
+
+class SchemaTabletsScanner : public SchemaScanner {
+ ENABLE_FACTORY_CREATOR(SchemaTabletsScanner)
+
public:
- static void insert_string_value(int col_index, std::string str_val,
vectorized::Block* block);
- static void insert_datetime_value(int col_index, const std::vector<void*>&
datas,
- vectorized::Block* block);
+ SchemaTabletsScanner();
- static void insert_int64_value(int col_index, int64_t int_val,
vectorized::Block* block);
- static void insert_double_value(int col_index, double double_val,
vectorized::Block* block);
-};
+ ~SchemaTabletsScanner() override = default;
+ Status start(RuntimeState* state) override;
+
+ Status get_next_block_internal(vectorized::Block* block, bool* eos)
override;
+
+private:
+ Status _get_all_tablets();
+
+ Status _fill_block_impl(vectorized::Block* block);
+
+ int64_t _backend_id {};
+ std::vector<BaseTabletSPtr> _tablets;
+ static std::vector<SchemaScanner::ColumnDesc> _s_tbls_columns;
+};
} // namespace doris
-#endif
diff --git a/be/src/olap/base_tablet.h b/be/src/olap/base_tablet.h
index 4df16de7eb3..9dd69d0bd9a 100644
--- a/be/src/olap/base_tablet.h
+++ b/be/src/olap/base_tablet.h
@@ -17,6 +17,8 @@
#pragma once
+#include <gen_cpp/olap_common.pb.h>
+
#include <memory>
#include <shared_mutex>
#include <string>
@@ -63,10 +65,12 @@ public:
int64_t partition_id() const { return _tablet_meta->partition_id(); }
int64_t tablet_id() const { return _tablet_meta->tablet_id(); }
int32_t schema_hash() const { return _tablet_meta->schema_hash(); }
+ size_t row_size() const { return
_tablet_meta->tablet_schema()->row_size(); }
KeysType keys_type() const { return
_tablet_meta->tablet_schema()->keys_type(); }
size_t num_key_columns() const { return
_tablet_meta->tablet_schema()->num_key_columns(); }
int64_t ttl_seconds() const { return _tablet_meta->ttl_seconds(); }
std::mutex& get_schema_change_lock() { return _schema_change_lock; }
+ CompressKind compress_kind() const { return
_tablet_meta->tablet_schema()->compress_kind(); }
bool enable_unique_key_merge_on_write() const {
#ifdef BE_TEST
if (_tablet_meta == nullptr) {
@@ -93,6 +97,11 @@ public:
return _max_version_schema;
}
+ void set_alter_failed(bool alter_failed) { _alter_failed = alter_failed; }
+ bool is_alter_failed() { return _alter_failed; }
+
+ virtual std::string tablet_path() const = 0;
+
virtual bool exceed_version_limit(int32_t limit) = 0;
virtual Result<std::unique_ptr<RowsetWriter>>
create_rowset_writer(RowsetWriterContext& context,
@@ -334,6 +343,9 @@ protected:
const TabletMetaSharedPtr _tablet_meta;
TabletSchemaSPtr _max_version_schema;
+ // `_alter_failed` is used to indicate whether the tablet failed to
perform a schema change
+ std::atomic<bool> _alter_failed = false;
+
// metrics of this tablet
std::shared_ptr<MetricEntity> _metric_entity;
diff --git a/be/src/olap/tablet.h b/be/src/olap/tablet.h
index 35993116bf6..467fc51f985 100644
--- a/be/src/olap/tablet.h
+++ b/be/src/olap/tablet.h
@@ -116,7 +116,7 @@ public:
DataDir* data_dir() const { return _data_dir; }
int64_t replica_id() const { return _tablet_meta->replica_id(); }
- const std::string& tablet_path() const { return _tablet_path; }
+ std::string tablet_path() const override { return _tablet_path; }
bool set_tablet_schema_into_rowset_meta();
Status init();
@@ -160,10 +160,8 @@ public:
size_t num_null_columns() const;
size_t num_short_key_columns() const;
size_t num_rows_per_row_block() const;
- CompressKind compress_kind() const;
double bloom_filter_fpp() const;
size_t next_unique_id() const;
- size_t row_size() const;
int64_t avg_rs_meta_serialize_size() const;
// operation in rowsets
@@ -484,9 +482,6 @@ public:
void set_binlog_config(BinlogConfig binlog_config);
- void set_alter_failed(bool alter_failed) { _alter_failed = alter_failed; }
- bool is_alter_failed() { return _alter_failed; }
-
void set_is_full_compaction_running(bool is_full_compaction_running) {
_is_full_compaction_running = is_full_compaction_running;
}
@@ -620,8 +615,6 @@ private:
// may delete compaction input rowsets.
std::mutex _cold_compaction_lock;
int64_t _last_failed_follow_cooldown_time = 0;
- // `_alter_failed` is used to indicate whether the tablet failed to
perform a schema change
- std::atomic<bool> _alter_failed = false;
int64_t _io_error_times = 0;
@@ -743,10 +736,6 @@ inline size_t Tablet::num_rows_per_row_block() const {
return _tablet_meta->tablet_schema()->num_rows_per_row_block();
}
-inline CompressKind Tablet::compress_kind() const {
- return _tablet_meta->tablet_schema()->compress_kind();
-}
-
inline double Tablet::bloom_filter_fpp() const {
return _tablet_meta->tablet_schema()->bloom_filter_fpp();
}
@@ -755,10 +744,6 @@ inline size_t Tablet::next_unique_id() const {
return _tablet_meta->tablet_schema()->next_column_unique_id();
}
-inline size_t Tablet::row_size() const {
- return _tablet_meta->tablet_schema()->row_size();
-}
-
inline int64_t Tablet::avg_rs_meta_serialize_size() const {
return _tablet_meta->avg_rs_meta_serialize_size();
}
diff --git
a/fe/fe-core/src/main/java/org/apache/doris/analysis/SchemaTableType.java
b/fe/fe-core/src/main/java/org/apache/doris/analysis/SchemaTableType.java
index 0ec8f6b851f..ea50de3f83f 100644
--- a/fe/fe-core/src/main/java/org/apache/doris/analysis/SchemaTableType.java
+++ b/fe/fe-core/src/main/java/org/apache/doris/analysis/SchemaTableType.java
@@ -67,6 +67,7 @@ public enum SchemaTableType {
SCH_CREATE_TABLE("CREATE_TABLE", "CREATE_TABLE",
TSchemaTableType.SCH_CREATE_TABLE),
SCH_INVALID("NULL", "NULL", TSchemaTableType.SCH_INVALID),
SCH_ROWSETS("ROWSETS", "ROWSETS", TSchemaTableType.SCH_ROWSETS),
+ SCH_TABLETS("BACKEND_TABLETS", "BACKEND_TABLETS",
TSchemaTableType.SCH_BACKEND_TABLETS),
SCH_PARAMETERS("PARAMETERS", "PARAMETERS",
TSchemaTableType.SCH_PARAMETERS),
SCH_METADATA_NAME_IDS("METADATA_NAME_IDS", "METADATA_NAME_IDS",
TSchemaTableType.SCH_METADATA_NAME_IDS),
SCH_PROFILING("PROFILING", "PROFILING", TSchemaTableType.SCH_PROFILING),
diff --git a/fe/fe-core/src/main/java/org/apache/doris/catalog/SchemaTable.java
b/fe/fe-core/src/main/java/org/apache/doris/catalog/SchemaTable.java
index 31a80d3daf6..ea8ccf8e787 100644
--- a/fe/fe-core/src/main/java/org/apache/doris/catalog/SchemaTable.java
+++ b/fe/fe-core/src/main/java/org/apache/doris/catalog/SchemaTable.java
@@ -477,7 +477,7 @@ public class SchemaTable extends Table {
.column("CURRENT_USED_MEMORY_BYTES",
ScalarType.createType(PrimitiveType.BIGINT))
.column("SHUFFLE_SEND_BYTES",
ScalarType.createType(PrimitiveType.BIGINT))
.column("SHUFFLE_SEND_ROWS",
ScalarType.createType(PrimitiveType.BIGINT))
- .column("QUERY_TYPE",
ScalarType.createVarchar(256))
+ .column("QUERY_TYPE",
ScalarType.createVarchar(256))
.build()))
.put("active_queries", new
SchemaTable(SystemIdGenerator.getNextId(), "active_queries", TableType.SCHEMA,
builder().column("QUERY_ID", ScalarType.createVarchar(256))
@@ -624,6 +624,27 @@ public class SchemaTable extends Table {
.column("IS_ABNORMAL_PAUSE",
ScalarType.createType(PrimitiveType.BOOLEAN))
.build())
)
+ .put("backend_tablets", new
SchemaTable(SystemIdGenerator.getNextId(), "backend_tablets", TableType.SCHEMA,
+ builder().column("BE_ID",
ScalarType.createType(PrimitiveType.BIGINT))
+ .column("TABLET_ID",
ScalarType.createType(PrimitiveType.BIGINT))
+ .column("REPLICA_ID",
ScalarType.createType(PrimitiveType.BIGINT))
+ .column("PARTITION_ID",
ScalarType.createType(PrimitiveType.BIGINT))
+ .column("TABLET_PATH",
ScalarType.createStringType())
+ .column("TABLET_LOCAL_SIZE",
ScalarType.createType(PrimitiveType.BIGINT))
+ .column("TABLET_REMOTE_SIZE",
ScalarType.createType(PrimitiveType.BIGINT))
+ .column("VERSION_COUNT",
ScalarType.createType(PrimitiveType.BIGINT))
+ .column("SEGMENT_COUNT",
ScalarType.createType(PrimitiveType.BIGINT))
+ .column("NUM_COLUMNS",
ScalarType.createType(PrimitiveType.BIGINT))
+ .column("ROW_SIZE",
ScalarType.createType(PrimitiveType.BIGINT))
+ .column("COMPACTION_SCORE",
ScalarType.createType(PrimitiveType.INT))
+ .column("COMPRESS_KIND",
ScalarType.createStringType())
+ .column("IS_USED",
ScalarType.createType(PrimitiveType.BOOLEAN))
+ .column("IS_ALTER_FAILED",
ScalarType.createType(PrimitiveType.BOOLEAN))
+ .column("CREATE_TIME",
ScalarType.createType(PrimitiveType.DATETIME))
+ .column("UPDATE_TIME",
ScalarType.createType(PrimitiveType.DATETIME))
+ .column("IS_OVERLAP",
ScalarType.createType(PrimitiveType.BOOLEAN))
+ .build())
+ )
.build();
private boolean fetchAllFe = false;
diff --git
a/fe/fe-core/src/main/java/org/apache/doris/planner/BackendPartitionedSchemaScanNode.java
b/fe/fe-core/src/main/java/org/apache/doris/planner/BackendPartitionedSchemaScanNode.java
index bc73cd8ce1b..8e96b04d3eb 100644
---
a/fe/fe-core/src/main/java/org/apache/doris/planner/BackendPartitionedSchemaScanNode.java
+++
b/fe/fe-core/src/main/java/org/apache/doris/planner/BackendPartitionedSchemaScanNode.java
@@ -70,6 +70,7 @@ public class BackendPartitionedSchemaScanNode extends
SchemaScanNode {
BEACKEND_ID_COLUMN_SET.add("be_id");
BACKEND_TABLE.add("file_cache_statistics");
+ BACKEND_TABLE.add("backend_tablets");
BACKEND_TABLE.add("backend_configuration");
}
diff --git a/gensrc/thrift/Descriptors.thrift b/gensrc/thrift/Descriptors.thrift
index 9c5e6f075b9..c76e64641dc 100644
--- a/gensrc/thrift/Descriptors.thrift
+++ b/gensrc/thrift/Descriptors.thrift
@@ -141,7 +141,8 @@ enum TSchemaTableType {
SCH_CATALOG_META_CACHE_STATISTICS = 52;
// consistent with the master
SCH_ROUTINE_LOAD_JOBS = 54,
- SCH_BACKEND_CONFIGURATION=55;
+ SCH_BACKEND_CONFIGURATION=55,
+ SCH_BACKEND_TABLETS = 56;
}
enum THdfsCompression {
diff --git
a/regression-test/data/external_table_p0/info_schema_db/backend_tablets.out
b/regression-test/data/external_table_p0/info_schema_db/backend_tablets.out
new file mode 100644
index 00000000000..6a79ef81689
Binary files /dev/null and
b/regression-test/data/external_table_p0/info_schema_db/backend_tablets.out
differ
diff --git
a/regression-test/suites/external_table_p0/info_schema_db/backend_tablets.groovy
b/regression-test/suites/external_table_p0/info_schema_db/backend_tablets.groovy
new file mode 100644
index 00000000000..cd194d958e9
--- /dev/null
+++
b/regression-test/suites/external_table_p0/info_schema_db/backend_tablets.groovy
@@ -0,0 +1,126 @@
+// 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("backend_tablets", "p0,
external_table,information_schema,backend_tablets") {
+ if (!isCloudMode()) {
+ def dbName = "test_backend_tablets_db"
+ def tbName1 = "test_backend_tablets_1"
+ def tbName2 = "test_backend_tablets_2"
+ def tbName3 = "test_backend_tablets_3"
+ sql(" drop database IF EXISTS ${dbName}")
+ sql(" create database ${dbName}")
+
+ sql("use ${dbName}")
+ sql("drop table IF EXISTS ${tbName1}")
+ sql("drop table IF EXISTS ${tbName2}")
+ sql("drop table IF EXISTS ${tbName3}")
+
+ sql """
+ CREATE TABLE IF NOT EXISTS `${tbName1}` (
+ `aaa` varchar(170) NOT NULL COMMENT "",
+ `bbb` varchar(20) NOT NULL COMMENT "",
+ `ccc` INT NULL COMMENT "",
+ `ddd` SMALLINT NULL COMMENT ""
+ )
+ DISTRIBUTED BY HASH(`aaa`) BUCKETS 1
+ PROPERTIES (
+ "replication_num" = "1"
+ );
+ """
+
+ sql """
+ CREATE TABLE IF NOT EXISTS `${tbName2}` (
+ `aaa` varchar(170) NOT NULL COMMENT "",
+ `bbb` string NOT NULL COMMENT "",
+ `ccc` INT NULL COMMENT "",
+ `ddd` SMALLINT NULL COMMENT ""
+ )
+ DISTRIBUTED BY HASH(`aaa`) BUCKETS 1
+ PROPERTIES (
+ "replication_num" = "1"
+ );
+ """
+
+ sql """
+ CREATE TABLE IF NOT EXISTS `${tbName3}` (
+ `aaa` varchar(170) NOT NULL COMMENT "",
+ `bbb` string NOT NULL COMMENT "",
+ `ccc` INT NULL COMMENT "",
+ `ddd` BIGINT NULL COMMENT ""
+ )
+ DISTRIBUTED BY HASH(`aaa`) BUCKETS 1
+ PROPERTIES (
+ "replication_num" = "1"
+ );
+ """
+
+ sql """
+ INSERT INTO `${tbName1}` (aaa, bbb, ccc, ddd) VALUES
+ ('value1', 'string1', 100, 10),
+ ('value2', 'string2', 200, 20),
+ ('value3', 'string3', 300, 30),
+ ('value4', 'string4', 400, 40),
+ ('value5', 'string5', 500, 50),
+ ('value6', 'string6', 600, 60),
+ ('value7', 'string7', 700, 70),
+ ('value8', 'string8', 800, 80),
+ ('value9', 'string9', 900, 90),
+ ('value10', 'string10', 1000, 100);
+ """
+
+ sql """
+ INSERT INTO `${tbName2}` (aaa, bbb, ccc, ddd) VALUES
+ ('value1', 'string1', 100, 10),
+ ('value2', 'string2', 200, 20),
+ ('value3', 'string3', 300, 30),
+ ('value4', 'string4', 400, 40),
+ ('value5', 'string5', 500, 50),
+ ('value6', 'string6', 600, 60),
+ ('value7', 'string7', 700, 70),
+ ('value8', 'string8', 800, 80),
+ ('value9', 'string9', 900, 90),
+ ('value10', 'string10', 1000, 100);
+ """
+
+ sql """
+ INSERT INTO `${tbName3}` (aaa, bbb, ccc, ddd) VALUES
+ ('value1', 'string1', 100, 10),
+ ('value2', 'string2', 200, 20),
+ ('value3', 'string3', 300, 30),
+ ('value4', 'string4', 400, 40),
+ ('value5', 'string5', 500, 50),
+ ('value6', 'string6', 600, 60),
+ ('value7', 'string7', 700, 70),
+ ('value8', 'string8', 800, 80),
+ ('value9', 'string9', 900, 90),
+ ('value10', 'string10', 1000, 100);
+ """
+
+ sql("use information_schema")
+ order_qt_1 """
+ SELECT
+ CASE
+ WHEN tablets_count.count_result >= 3 THEN 'true'
+ ELSE 'false'
+ END AS result
+ FROM
+ (SELECT COUNT(*) AS count_result FROM backend_tablets) AS
tablets_count;
+ """
+
+ }
+}
+
---------------------------------------------------------------------
To unsubscribe, e-mail: [email protected]
For additional commands, e-mail: [email protected]