You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@doris.apache.org by mo...@apache.org on 2022/04/27 02:53:48 UTC

[incubator-doris] branch master updated: [feature] add `SHOW TABLET STORAGE FORMAT` stmt (#9037)

This is an automated email from the ASF dual-hosted git repository.

morningman pushed a commit to branch master
in repository https://gitbox.apache.org/repos/asf/incubator-doris.git


The following commit(s) were added to refs/heads/master by this push:
     new 597115c305 [feature] add `SHOW TABLET STORAGE FORMAT` stmt (#9037)
597115c305 is described below

commit 597115c305fcdbc6ec0a8caa70b44cf037f6dcc1
Author: Zhengguo Yang <ya...@gmail.com>
AuthorDate: Wed Apr 27 10:53:43 2022 +0800

    [feature] add `SHOW TABLET STORAGE FORMAT` stmt (#9037)
    
    use this stmt to show tablets storage format in be, if verbose is set,
        will show detail message of tablet storage format.
        e.g.
        ```
        MySQL [(none)]> admin show tablet storage format;
        +-----------+---------+---------+
        | BackendId | V1Count | V2Count |
        +-----------+---------+---------+
        | 10002     | 0       | 2867    |
        +-----------+---------+---------+
        1 row in set (0.003 sec)
        MySQL [test_query_qa]> admin show tablet storage format verbose;
        +-----------+----------+---------------+
        | BackendId | TabletId | StorageFormat |
        +-----------+----------+---------------+
        | 10002     | 39227    | V2            |
        | 10002     | 39221    | V2            |
        | 10002     | 39215    | V2            |
        | 10002     | 39199    | V2            |
        +-----------+----------+---------------+
        4 rows in set (0.034 sec)
        ```
        add storage format infomation to show full table statment.
        ```
        MySQL [test_query_qa]> show full tables;
        +-------------------------+------------+---------------+
        | Tables_in_test_query_qa | Table_type | StorageFormat |
        +-------------------------+------------+---------------+
        | bigtable                | BASE TABLE | V2            |
        | test_dup                | BASE TABLE | V2            |
        | test                    | BASE TABLE | V2            |
        | baseall                 | BASE TABLE | V2            |
        | test_string             | BASE TABLE | V2            |
        +-------------------------+------------+---------------+
        5 rows in set (0.002 sec)
        ```
---
 be/src/olap/tablet_manager.cpp                     | 98 +++++++++++++---------
 be/src/olap/tablet_manager.h                       | 46 +++++-----
 be/src/service/backend_service.cpp                 |  4 +
 be/src/service/backend_service.h                   |  2 +
 docs/.vuepress/sidebar/en.js                       |  1 +
 docs/.vuepress/sidebar/zh-CN.js                    |  1 +
 .../ADMIN SHOW TABLET STORAGE FORMAT.md            | 53 ++++++++++++
 .../ADMIN SHOW TABLET STORAGE FORMAT.md            | 53 ++++++++++++
 fe/fe-core/src/main/cup/sql_parser.cup             |  8 ++
 .../analysis/AdminShowTabletStorageFormatStmt.java | 83 ++++++++++++++++++
 .../org/apache/doris/analysis/ShowTableStmt.java   |  2 +
 .../java/org/apache/doris/qe/ShowExecutor.java     | 57 +++++++++++--
 .../java/org/apache/doris/task/AgentClient.java    | 17 ++++
 .../apache/doris/analysis/ShowTableStmtTest.java   |  6 +-
 .../org/apache/doris/common/GenericPoolTest.java   |  6 ++
 .../apache/doris/utframe/MockedBackendFactory.java |  6 ++
 gensrc/thrift/BackendService.thrift                |  8 ++
 17 files changed, 381 insertions(+), 70 deletions(-)

diff --git a/be/src/olap/tablet_manager.cpp b/be/src/olap/tablet_manager.cpp
index 4e8019e5c0..4664cb7d35 100644
--- a/be/src/olap/tablet_manager.cpp
+++ b/be/src/olap/tablet_manager.cpp
@@ -89,7 +89,7 @@ TabletManager::~TabletManager() {
     DEREGISTER_HOOK_METRIC(tablet_meta_mem_consumption);
 }
 
-Status TabletManager::_add_tablet_unlocked(TTabletId tablet_id, const TabletSharedPtr& tablet, 
+Status TabletManager::_add_tablet_unlocked(TTabletId tablet_id, const TabletSharedPtr& tablet,
                                            bool update_meta, bool force) {
     Status res = Status::OK();
     VLOG_NOTICE << "begin to add tablet to TabletManager. "
@@ -174,9 +174,8 @@ Status TabletManager::_add_tablet_unlocked(TTabletId tablet_id, const TabletShar
 }
 
 Status TabletManager::_add_tablet_to_map_unlocked(TTabletId tablet_id,
-                                                      const TabletSharedPtr& tablet,
-                                                      bool update_meta, bool keep_files,
-                                                      bool drop_old) {
+                                                  const TabletSharedPtr& tablet, bool update_meta,
+                                                  bool keep_files, bool drop_old) {
     // check if new tablet's meta is in store and add new tablet's meta to meta store
     Status res = Status::OK();
     if (update_meta) {
@@ -215,8 +214,7 @@ bool TabletManager::_check_tablet_id_exist_unlocked(TTabletId tablet_id) {
     return tablet_map.find(tablet_id) != tablet_map.end();
 }
 
-Status TabletManager::create_tablet(const TCreateTabletReq& request,
-                                        std::vector<DataDir*> stores) {
+Status TabletManager::create_tablet(const TCreateTabletReq& request, std::vector<DataDir*> stores) {
     SCOPED_SWITCH_THREAD_LOCAL_MEM_TRACKER(_mem_tracker);
     DorisMetrics::instance()->create_tablet_requests_total->increment(1);
 
@@ -414,8 +412,8 @@ TabletSharedPtr TabletManager::_create_tablet_meta_and_dir_unlocked(
 
         TabletMetaSharedPtr tablet_meta;
         // if create meta failed, do not need to clean dir, because it is only in memory
-        Status res = _create_tablet_meta_unlocked(request, data_dir, is_schema_change,
-                                                      base_tablet, &tablet_meta);
+        Status res = _create_tablet_meta_unlocked(request, data_dir, is_schema_change, base_tablet,
+                                                  &tablet_meta);
         if (!res.ok()) {
             LOG(WARNING) << "fail to create tablet meta. res=" << res
                          << ", root=" << data_dir->path();
@@ -443,12 +441,15 @@ TabletSharedPtr TabletManager::_create_tablet_meta_and_dir_unlocked(
         }
 
         StorageParamPB storage_param;
-        Status status = _get_storage_param(data_dir, tablet_meta->remote_storage_name(), &storage_param);
+        Status status =
+                _get_storage_param(data_dir, tablet_meta->remote_storage_name(), &storage_param);
         if (!status.ok()) {
-            LOG(WARNING) << "fail to _get_storage_param. storage_name: " << tablet_meta->remote_storage_name();
+            LOG(WARNING) << "fail to _get_storage_param. storage_name: "
+                         << tablet_meta->remote_storage_name();
             return nullptr;
         }
-        TabletSharedPtr new_tablet = Tablet::create_tablet_from_meta(tablet_meta, storage_param, data_dir);
+        TabletSharedPtr new_tablet =
+                Tablet::create_tablet_from_meta(tablet_meta, storage_param, data_dir);
         DCHECK(new_tablet != nullptr);
         return new_tablet;
     }
@@ -557,7 +558,8 @@ TabletSharedPtr TabletManager::_get_tablet_unlocked(TTabletId tablet_id, bool in
     return tablet;
 }
 
-TabletSharedPtr TabletManager::get_tablet(TTabletId tablet_id, TabletUid tablet_uid, bool include_deleted, string* err) {
+TabletSharedPtr TabletManager::get_tablet(TTabletId tablet_id, TabletUid tablet_uid,
+                                          bool include_deleted, string* err) {
     std::shared_lock rdlock(_get_tablets_shard_lock(tablet_id));
     TabletSharedPtr tablet = _get_tablet_unlocked(tablet_id, include_deleted, err);
     if (tablet != nullptr && tablet->tablet_uid() == tablet_uid) {
@@ -694,9 +696,9 @@ TabletSharedPtr TabletManager::find_best_tablet_to_compaction(
 }
 
 Status TabletManager::load_tablet_from_meta(DataDir* data_dir, TTabletId tablet_id,
-                                                TSchemaHash schema_hash, const string& meta_binary,
-                                                bool update_meta, bool force, bool restore,
-                                                bool check_path) {
+                                            TSchemaHash schema_hash, const string& meta_binary,
+                                            bool update_meta, bool force, bool restore,
+                                            bool check_path) {
     SCOPED_SWITCH_THREAD_LOCAL_MEM_TRACKER(_mem_tracker);
     TabletMetaSharedPtr tablet_meta(new TabletMeta());
     Status status = tablet_meta->deserialize(meta_binary);
@@ -728,8 +730,9 @@ Status TabletManager::load_tablet_from_meta(DataDir* data_dir, TTabletId tablet_
     }
 
     StorageParamPB storage_param;
-    RETURN_NOT_OK_LOG(_get_storage_param(data_dir, tablet_meta->remote_storage_name(), &storage_param),
-                      "fail to _get_storage_param. storage_name: " + tablet_meta->remote_storage_name());
+    RETURN_NOT_OK_LOG(
+            _get_storage_param(data_dir, tablet_meta->remote_storage_name(), &storage_param),
+            "fail to _get_storage_param. storage_name: " + tablet_meta->remote_storage_name());
 
     TabletSharedPtr tablet = Tablet::create_tablet_from_meta(tablet_meta, storage_param, data_dir);
     if (tablet == nullptr) {
@@ -781,9 +784,8 @@ Status TabletManager::load_tablet_from_meta(DataDir* data_dir, TTabletId tablet_
 }
 
 Status TabletManager::load_tablet_from_dir(DataDir* store, TTabletId tablet_id,
-                                               SchemaHash schema_hash,
-                                               const string& schema_hash_path, bool force,
-                                               bool restore) {
+                                           SchemaHash schema_hash, const string& schema_hash_path,
+                                           bool force, bool restore) {
     SCOPED_SWITCH_THREAD_LOCAL_MEM_TRACKER(_mem_tracker);
     LOG(INFO) << "begin to load tablet from dir. "
               << " tablet_id=" << tablet_id << " schema_hash=" << schema_hash
@@ -842,8 +844,7 @@ Status TabletManager::report_tablet_info(TTabletInfo* tablet_info) {
     return res;
 }
 
-Status TabletManager::build_all_report_tablets_info(
-        std::map<TTabletId, TTablet>* tablets_info) {
+Status TabletManager::build_all_report_tablets_info(std::map<TTabletId, TTablet>* tablets_info) {
     DCHECK(tablets_info != nullptr);
     LOG(INFO) << "begin to build all report tablets info";
 
@@ -931,7 +932,7 @@ Status TabletManager::start_trash_sweep() {
             }
             TabletMetaSharedPtr tablet_meta(new TabletMeta());
             Status check_st = TabletMetaManager::get_meta((*it)->data_dir(), (*it)->tablet_id(),
-                                                              (*it)->schema_hash(), tablet_meta);
+                                                          (*it)->schema_hash(), tablet_meta);
             if (check_st.ok()) {
                 if (tablet_meta->tablet_state() != TABLET_SHUTDOWN ||
                     tablet_meta->tablet_uid() != (*it)->tablet_uid()) {
@@ -951,10 +952,12 @@ Status TabletManager::start_trash_sweep() {
                     string meta_file_path = path_util::join_path_segments(
                             tablet_path_desc.filepath, std::to_string((*it)->tablet_id()) + ".hdr");
                     (*it)->tablet_meta()->save(meta_file_path);
-                    LOG(INFO) << "start to move tablet to trash. " << tablet_path_desc.debug_string();
+                    LOG(INFO) << "start to move tablet to trash. "
+                              << tablet_path_desc.debug_string();
                     Status rm_st = (*it)->data_dir()->move_to_trash(tablet_path_desc);
                     if (rm_st != Status::OK()) {
-                        LOG(WARNING) << "fail to move dir to trash. " << tablet_path_desc.debug_string();
+                        LOG(WARNING)
+                                << "fail to move dir to trash. " << tablet_path_desc.debug_string();
                         ++it;
                         continue;
                     }
@@ -1018,8 +1021,7 @@ void TabletManager::try_delete_unused_tablet_path(DataDir* data_dir, TTabletId t
 
     // check if meta already exists
     TabletMetaSharedPtr tablet_meta(new TabletMeta());
-    Status check_st =
-            TabletMetaManager::get_meta(data_dir, tablet_id, schema_hash, tablet_meta);
+    Status check_st = TabletMetaManager::get_meta(data_dir, tablet_id, schema_hash, tablet_meta);
     if (check_st.ok()) {
         LOG(INFO) << "tablet meta exist is meta store, skip delete the path " << schema_hash_path;
         return;
@@ -1147,7 +1149,7 @@ void TabletManager::do_tablet_meta_checkpoint(DataDir* data_dir) {
 }
 
 Status TabletManager::_create_initial_rowset_unlocked(const TCreateTabletReq& request,
-                                                          Tablet* tablet) {
+                                                      Tablet* tablet) {
     Status res = Status::OK();
     if (request.version < 1) {
         LOG(WARNING) << "init version of tablet should at least 1. req.ver=" << request.version;
@@ -1216,10 +1218,10 @@ Status TabletManager::_create_initial_rowset_unlocked(const TCreateTabletReq& re
     return res;
 }
 
-Status TabletManager::_create_tablet_meta_unlocked(const TCreateTabletReq& request,
-                                                       DataDir* store, const bool is_schema_change,
-                                                       const Tablet* base_tablet,
-                                                       TabletMetaSharedPtr* tablet_meta) {
+Status TabletManager::_create_tablet_meta_unlocked(const TCreateTabletReq& request, DataDir* store,
+                                                   const bool is_schema_change,
+                                                   const Tablet* base_tablet,
+                                                   TabletMetaSharedPtr* tablet_meta) {
     uint32_t next_unique_id = 0;
     std::unordered_map<uint32_t, uint32_t> col_idx_to_unique_id;
     if (!is_schema_change) {
@@ -1254,7 +1256,7 @@ Status TabletManager::_create_tablet_meta_unlocked(const TCreateTabletReq& reque
     uint64_t shard_id = 0;
     RETURN_NOT_OK_LOG(store->get_shard(&shard_id), "fail to get root path shard");
     Status res = TabletMeta::create(request, TabletUid::gen_uid(), shard_id, next_unique_id,
-                                        col_idx_to_unique_id, tablet_meta);
+                                    col_idx_to_unique_id, tablet_meta);
 
     if (request.__isset.storage_format && request.storage_format != TStorageFormat::V1) {
         (*tablet_meta)->set_preferred_rowset_type(BETA_ROWSET);
@@ -1388,15 +1390,35 @@ void TabletManager::get_tablets_distribution_on_different_disks(
     }
 }
 
-Status TabletManager::_get_storage_param(
-        DataDir* data_dir, const std::string& storage_name, StorageParamPB* storage_param) {
+Status TabletManager::_get_storage_param(DataDir* data_dir, const std::string& storage_name,
+                                         StorageParamPB* storage_param) {
     if (data_dir->is_remote()) {
-        RETURN_WITH_WARN_IF_ERROR(StorageBackendMgr::instance()->get_storage_param(storage_name, storage_param),
-                                  Status::OLAPInternalError(OLAP_ERR_OTHER_ERROR), "get_storage_param failed for storage_name: " + storage_name);
+        RETURN_WITH_WARN_IF_ERROR(
+                StorageBackendMgr::instance()->get_storage_param(storage_name, storage_param),
+                Status::OLAPInternalError(OLAP_ERR_OTHER_ERROR),
+                "get_storage_param failed for storage_name: " + storage_name);
     } else {
-        storage_param->set_storage_medium(fs::fs_util::get_storage_medium_pb(data_dir->storage_medium()));
+        storage_param->set_storage_medium(
+                fs::fs_util::get_storage_medium_pb(data_dir->storage_medium()));
     }
     return Status::OK();
 }
 
+void TabletManager::get_all_tablets_storage_format(TCheckStorageFormatResult* result) {
+    DCHECK(result != nullptr);
+    for (const auto& tablets_shard : _tablets_shards) {
+        std::shared_lock rdlock(tablets_shard.lock);
+        for (const auto& item : tablets_shard.tablet_map) {
+            uint64_t tablet_id = item.first;
+            if (item.second->all_beta()) {
+                result->v2_tablets.push_back(tablet_id);
+            } else {
+                result->v1_tablets.push_back(tablet_id);
+            }
+        }
+    }
+    result->__isset.v1_tablets = true;
+    result->__isset.v2_tablets = true;
+}
+
 } // end namespace doris
diff --git a/be/src/olap/tablet_manager.h b/be/src/olap/tablet_manager.h
index 2fc565e39a..c7c4d8990d 100644
--- a/be/src/olap/tablet_manager.h
+++ b/be/src/olap/tablet_manager.h
@@ -75,11 +75,12 @@ public:
             const std::unordered_set<TTabletId>& tablet_submitted_compaction, uint32_t* score,
             std::shared_ptr<CumulativeCompactionPolicy> cumulative_compaction_policy);
 
-    TabletSharedPtr get_tablet(TTabletId tablet_id, bool include_deleted = false, std::string* err = nullptr);
-
-    TabletSharedPtr get_tablet(TTabletId tablet_id, TabletUid tablet_uid, bool include_deleted = false,
+    TabletSharedPtr get_tablet(TTabletId tablet_id, bool include_deleted = false,
                                std::string* err = nullptr);
 
+    TabletSharedPtr get_tablet(TTabletId tablet_id, TabletUid tablet_uid,
+                               bool include_deleted = false, std::string* err = nullptr);
+
     // Extract tablet_id and schema_hash from given path.
     //
     // The normal path pattern is like "/data/{shard_id}/{tablet_id}/{schema_hash}/xxx.data".
@@ -100,14 +101,13 @@ public:
     // parse tablet header msg to generate tablet object
     // - restore: whether the request is from restore tablet action,
     //   where we should change tablet status from shutdown back to running
-    Status load_tablet_from_meta(DataDir* data_dir, TTabletId tablet_id,
-                                     TSchemaHash schema_hash, const std::string& header,
-                                     bool update_meta, bool force = false, bool restore = false,
-                                     bool check_path = true);
+    Status load_tablet_from_meta(DataDir* data_dir, TTabletId tablet_id, TSchemaHash schema_hash,
+                                 const std::string& header, bool update_meta, bool force = false,
+                                 bool restore = false, bool check_path = true);
 
     Status load_tablet_from_dir(DataDir* data_dir, TTabletId tablet_id, SchemaHash schema_hash,
-                                    const std::string& schema_hash_path, bool force = false,
-                                    bool restore = false);
+                                const std::string& schema_hash_path, bool force = false,
+                                bool restore = false);
 
     // 获取所有tables的名字
     //
@@ -138,6 +138,8 @@ public:
             std::map<int64_t, std::map<DataDir*, int64_t>>& tablets_num_on_disk,
             std::map<int64_t, std::map<DataDir*, std::vector<TabletSize>>>& tablets_info_on_disk);
 
+    void get_all_tablets_storage_format(TCheckStorageFormatResult* result);
+
 private:
     // Add a tablet pointer to StorageEngine
     // If force, drop the existing tablet add this new one
@@ -145,11 +147,11 @@ private:
     // Return OLAP_SUCCESS, if run ok
     //        OLAP_ERR_TABLE_INSERT_DUPLICATION_ERROR, if find duplication
     //        Status::OLAPInternalError(OLAP_ERR_NOT_INITED), if not inited
-    Status _add_tablet_unlocked(TTabletId tablet_id, const TabletSharedPtr& tablet, bool update_meta, bool force);
+    Status _add_tablet_unlocked(TTabletId tablet_id, const TabletSharedPtr& tablet,
+                                bool update_meta, bool force);
 
-    Status _add_tablet_to_map_unlocked(TTabletId tablet_id, 
-                                           const TabletSharedPtr& tablet, bool update_meta,
-                                           bool keep_files, bool drop_old);
+    Status _add_tablet_to_map_unlocked(TTabletId tablet_id, const TabletSharedPtr& tablet,
+                                       bool update_meta, bool keep_files, bool drop_old);
 
     bool _check_tablet_id_exist_unlocked(TTabletId tablet_id);
     Status _create_initial_rowset_unlocked(const TCreateTabletReq& request, Tablet* tablet);
@@ -159,7 +161,8 @@ private:
     Status _drop_tablet_unlocked(TTabletId tablet_id, bool keep_files);
 
     TabletSharedPtr _get_tablet_unlocked(TTabletId tablet_id);
-    TabletSharedPtr _get_tablet_unlocked(TTabletId tablet_id, bool include_deleted, std::string* err);
+    TabletSharedPtr _get_tablet_unlocked(TTabletId tablet_id, bool include_deleted,
+                                         std::string* err);
 
     TabletSharedPtr _internal_create_tablet_unlocked(const TCreateTabletReq& request,
                                                      const bool is_schema_change,
@@ -170,17 +173,19 @@ private:
                                                          const Tablet* base_tablet,
                                                          const std::vector<DataDir*>& data_dirs);
     Status _create_tablet_meta_unlocked(const TCreateTabletReq& request, DataDir* store,
-                                            const bool is_schema_change_tablet,
-                                            const Tablet* base_tablet,
-                                            TabletMetaSharedPtr* tablet_meta);
-    
+                                        const bool is_schema_change_tablet,
+                                        const Tablet* base_tablet,
+                                        TabletMetaSharedPtr* tablet_meta);
+
     void _add_tablet_to_partition(const TabletSharedPtr& tablet);
 
     void _remove_tablet_from_partition(const TabletSharedPtr& tablet);
 
     std::shared_mutex& _get_tablets_shard_lock(TTabletId tabletId);
 
-    Status _get_storage_param(DataDir* data_dir, const std::string& storage_name, StorageParamPB* storage_param);
+    Status _get_storage_param(DataDir* data_dir, const std::string& storage_name,
+                              StorageParamPB* storage_param);
+
 private:
     DISALLOW_COPY_AND_ASSIGN(TabletManager);
 
@@ -214,7 +219,8 @@ private:
     std::vector<TabletSharedPtr> _shutdown_tablets;
 
     std::mutex _tablet_stat_cache_mutex;
-    std::shared_ptr<std::vector<TTabletStat>> _tablet_stat_list_cache = std::make_shared<std::vector<TTabletStat>>();
+    std::shared_ptr<std::vector<TTabletStat>> _tablet_stat_list_cache =
+            std::make_shared<std::vector<TTabletStat>>();
 
     tablet_map_t& _get_tablet_map(TTabletId tablet_id);
 
diff --git a/be/src/service/backend_service.cpp b/be/src/service/backend_service.cpp
index 05eac43034..97e35e1597 100644
--- a/be/src/service/backend_service.cpp
+++ b/be/src/service/backend_service.cpp
@@ -370,4 +370,8 @@ void BackendService::get_stream_load_record(TStreamLoadRecordResult& result,
 void BackendService::clean_trash() {
     StorageEngine::instance()->start_trash_sweep(nullptr, true);
 }
+
+void BackendService::check_storage_format(TCheckStorageFormatResult& result) {
+    StorageEngine::instance()->tablet_manager()->get_all_tablets_storage_format(&result);
+}
 } // namespace doris
diff --git a/be/src/service/backend_service.h b/be/src/service/backend_service.h
index 3c9b3bd13c..7991f59c9f 100644
--- a/be/src/service/backend_service.h
+++ b/be/src/service/backend_service.h
@@ -156,6 +156,8 @@ public:
 
     virtual void clean_trash() override;
 
+    virtual void check_storage_format(TCheckStorageFormatResult& result) override;
+
 private:
     Status start_plan_fragment_execution(const TExecPlanFragmentParams& exec_params);
     ExecEnv* _exec_env;
diff --git a/docs/.vuepress/sidebar/en.js b/docs/.vuepress/sidebar/en.js
index 6a3d7ffda3..5f960c8f3b 100644
--- a/docs/.vuepress/sidebar/en.js
+++ b/docs/.vuepress/sidebar/en.js
@@ -593,6 +593,7 @@ module.exports = [
               "ADMIN SHOW REPLICA DISTRIBUTION",
               "ADMIN SHOW REPLICA STATUS",
               "ADMIN-DIAGNOSE-TABLET",
+              "ADMIN SHOW TABLET STORAGE FORMAT",
               "ALTER CLUSTER",
               "ALTER SYSTEM",
               "CANCEL DECOMMISSION",
diff --git a/docs/.vuepress/sidebar/zh-CN.js b/docs/.vuepress/sidebar/zh-CN.js
index 13ef8c0daf..324b9a7809 100644
--- a/docs/.vuepress/sidebar/zh-CN.js
+++ b/docs/.vuepress/sidebar/zh-CN.js
@@ -607,6 +607,7 @@ module.exports = [
               "ADMIN SHOW REPLICA DISTRIBUTION",
               "ADMIN SHOW REPLICA STATUS",
               "ADMIN-DIAGNOSE-TABLET",
+              "ADMIN SHOW TABLET STORAGE FORMAT",
               "ALTER CLUSTER",
               "ALTER SYSTEM",
               "CANCEL DECOMMISSION",
diff --git a/docs/en/sql-reference/sql-statements/Administration/ADMIN SHOW TABLET STORAGE FORMAT.md b/docs/en/sql-reference/sql-statements/Administration/ADMIN SHOW TABLET STORAGE FORMAT.md
new file mode 100644
index 0000000000..06692f0878
--- /dev/null
+++ b/docs/en/sql-reference/sql-statements/Administration/ADMIN SHOW TABLET STORAGE FORMAT.md	
@@ -0,0 +1,53 @@
+---
+{
+    "title": "ADMIN SHOW TABLET STORAGE FORMAT",
+    "language": "en"
+}
+---
+
+<!-- 
+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.
+-->
+
+# ADMIN SHOW TABLET STORAGE FORMAT
+## description
+    This statement is used to display tablet storage foramt information (for administrators only)
+    Grammar:
+        ADMIN SHOW TABLET STORAGE FORMAT [VERBOSE]
+
+## example
+    MySQL [(none)]> admin show tablet storage format;
+    +-----------+---------+---------+
+    | BackendId | V1Count | V2Count |
+    +-----------+---------+---------+
+    | 10002     | 0       | 2867    |
+    +-----------+---------+---------+
+    1 row in set (0.003 sec)
+    MySQL [test_query_qa]> admin show tablet storage format verbose;
+    +-----------+----------+---------------+
+    | BackendId | TabletId | StorageFormat |
+    +-----------+----------+---------------+
+    | 10002     | 39227    | V2            |
+    | 10002     | 39221    | V2            |
+    | 10002     | 39215    | V2            |
+    | 10002     | 39199    | V2            |
+    +-----------+----------+---------------+
+    4 rows in set (0.034 sec)
+## keyword
+    ADMIN,SHOW,TABLET,STORAGE,FORMAT
+
diff --git a/docs/zh-CN/sql-reference/sql-statements/Administration/ADMIN SHOW TABLET STORAGE FORMAT.md b/docs/zh-CN/sql-reference/sql-statements/Administration/ADMIN SHOW TABLET STORAGE FORMAT.md
new file mode 100644
index 0000000000..c7ff344012
--- /dev/null
+++ b/docs/zh-CN/sql-reference/sql-statements/Administration/ADMIN SHOW TABLET STORAGE FORMAT.md	
@@ -0,0 +1,53 @@
+---
+{
+    "title": "ADMIN SHOW TABLET STORAGE FORMAT",
+    "language": "zh-CN"
+}
+---
+
+<!-- 
+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.
+-->
+
+# ADMIN SHOW TABLET STORAGE FORMAT
+## description
+    该语句用于显示Backend上的存储格式信息(仅管理员使用)
+    语法:
+        ADMIN SHOW TABLET STORAGE FORMAT [VERBOSE]
+
+## example
+    MySQL [(none)]> admin show tablet storage format;
+    +-----------+---------+---------+
+    | BackendId | V1Count | V2Count |
+    +-----------+---------+---------+
+    | 10002     | 0       | 2867    |
+    +-----------+---------+---------+
+    1 row in set (0.003 sec)
+    MySQL [test_query_qa]> admin show tablet storage format verbose;
+    +-----------+----------+---------------+
+    | BackendId | TabletId | StorageFormat |
+    +-----------+----------+---------------+
+    | 10002     | 39227    | V2            |
+    | 10002     | 39221    | V2            |
+    | 10002     | 39215    | V2            |
+    | 10002     | 39199    | V2            |
+    +-----------+----------+---------------+
+    4 rows in set (0.034 sec)
+## keyword
+    ADMIN,SHOW,TABLET,STORAGE,FORMAT
+
diff --git a/fe/fe-core/src/main/cup/sql_parser.cup b/fe/fe-core/src/main/cup/sql_parser.cup
index cd71721636..27c88217b8 100644
--- a/fe/fe-core/src/main/cup/sql_parser.cup
+++ b/fe/fe-core/src/main/cup/sql_parser.cup
@@ -5332,6 +5332,14 @@ admin_stmt ::=
     {:
         RESULT = new AdminDiagnoseTabletStmt(tabletId);
     :}
+    | KW_ADMIN KW_TABLET KW_STORAGE KW_FORMAT
+    {:
+        RESULT = new AdminShowTabletStorageFormatStmt(false);
+    :}
+    | KW_ADMIN KW_TABLET KW_STORAGE KW_FORMAT KW_VERBOSE
+    {:
+        RESULT = new AdminShowTabletStorageFormatStmt(true);
+    :}
     ;
 
 truncate_stmt ::=
diff --git a/fe/fe-core/src/main/java/org/apache/doris/analysis/AdminShowTabletStorageFormatStmt.java b/fe/fe-core/src/main/java/org/apache/doris/analysis/AdminShowTabletStorageFormatStmt.java
new file mode 100644
index 0000000000..975731a5fd
--- /dev/null
+++ b/fe/fe-core/src/main/java/org/apache/doris/analysis/AdminShowTabletStorageFormatStmt.java
@@ -0,0 +1,83 @@
+// 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.
+
+package org.apache.doris.analysis;
+
+import org.apache.doris.catalog.Catalog;
+import org.apache.doris.catalog.Column;
+import org.apache.doris.catalog.ScalarType;
+import org.apache.doris.common.ErrorCode;
+import org.apache.doris.common.ErrorReport;
+import org.apache.doris.common.UserException;
+import org.apache.doris.mysql.privilege.PrivPredicate;
+import org.apache.doris.qe.ConnectContext;
+import org.apache.doris.qe.ShowResultSetMetaData;
+
+public class AdminShowTabletStorageFormatStmt extends ShowStmt {
+    private boolean verbose;
+
+    @Override
+    public boolean isVerbose() {
+        return verbose;
+    }
+
+    public AdminShowTabletStorageFormatStmt(boolean verbose) {
+        this.verbose = verbose;
+    }
+
+    @Override
+    public void analyze(Analyzer analyzer) throws UserException {
+        // check access first
+        if (!Catalog.getCurrentCatalog().getAuth().checkGlobalPriv(ConnectContext.get(), PrivPredicate.ADMIN)) {
+            ErrorReport.reportAnalysisException(ErrorCode.ERR_ACCESS_DENIED_ERROR,
+                    toSql(),
+                    ConnectContext.get().getQualifiedUser(),
+                    ConnectContext.get().getRemoteIP(), "ADMIN Privilege needed.");
+        }
+
+        super.analyze(analyzer);
+    }
+
+    @Override
+    public String toSql() {
+        StringBuilder sb = new StringBuilder("ADMIN SHOW TABLET STORAGE TYPE");
+        if (verbose) {
+            sb.append(" VERBOSE");
+        }
+        return sb.toString();
+    }
+
+    @Override
+    public String toString() {
+        return toSql();
+    }
+
+    @Override
+    public ShowResultSetMetaData getMetaData() {
+        ShowResultSetMetaData.Builder builder = ShowResultSetMetaData.builder();
+        if (verbose) {
+            builder.addColumn(new Column("BackendId", ScalarType.createVarchar(30)))
+                    .addColumn(new Column("TabletId", ScalarType.createVarchar(30)))
+                    .addColumn(new Column("StorageFormat", ScalarType.createVarchar(30)));
+        } else {
+            builder.addColumn(new Column("BackendId", ScalarType.createVarchar(30)))
+                    .addColumn(new Column("V1Count", ScalarType.createVarchar(30)))
+                    .addColumn(new Column("V2Count", ScalarType.createVarchar(30)));
+        }
+        return builder.build();
+    }
+}
diff --git a/fe/fe-core/src/main/java/org/apache/doris/analysis/ShowTableStmt.java b/fe/fe-core/src/main/java/org/apache/doris/analysis/ShowTableStmt.java
index f81dc9d0d6..41dd44adfd 100644
--- a/fe/fe-core/src/main/java/org/apache/doris/analysis/ShowTableStmt.java
+++ b/fe/fe-core/src/main/java/org/apache/doris/analysis/ShowTableStmt.java
@@ -37,6 +37,7 @@ public class ShowTableStmt extends ShowStmt {
     private static final Logger LOG = LogManager.getLogger(ShowTableStmt.class);
     private static final String NAME_COL_PREFIX = "Tables_in_";
     private static final String TYPE_COL = "Table_type";
+    private static final String STORAGE_FORMAT_COL = "StorageFormat";
     private static final TableName TABLE_NAME = new TableName(InfoSchemaDb.DATABASE_NAME, "tables");
     private String db;
     private boolean isVerbose;
@@ -146,6 +147,7 @@ public class ShowTableStmt extends ShowStmt {
                 new Column(NAME_COL_PREFIX + ClusterNamespace.getNameFromFullName(db), ScalarType.createVarchar(20)));
         if (isVerbose) {
             builder.addColumn(new Column(TYPE_COL, ScalarType.createVarchar(20)));
+            builder.addColumn(new Column(STORAGE_FORMAT_COL, ScalarType.createVarchar(20)));
         }
         return builder.build();
     }
diff --git a/fe/fe-core/src/main/java/org/apache/doris/qe/ShowExecutor.java b/fe/fe-core/src/main/java/org/apache/doris/qe/ShowExecutor.java
index 3397d018fe..4decb80a00 100644
--- a/fe/fe-core/src/main/java/org/apache/doris/qe/ShowExecutor.java
+++ b/fe/fe-core/src/main/java/org/apache/doris/qe/ShowExecutor.java
@@ -21,6 +21,7 @@ import org.apache.doris.analysis.AdminDiagnoseTabletStmt;
 import org.apache.doris.analysis.AdminShowConfigStmt;
 import org.apache.doris.analysis.AdminShowReplicaDistributionStmt;
 import org.apache.doris.analysis.AdminShowReplicaStatusStmt;
+import org.apache.doris.analysis.AdminShowTabletStorageFormatStmt;
 import org.apache.doris.analysis.DescribeStmt;
 import org.apache.doris.analysis.HelpStmt;
 import org.apache.doris.analysis.PartitionNames;
@@ -158,6 +159,8 @@ import org.apache.doris.mysql.privilege.PrivPredicate;
 import org.apache.doris.system.Backend;
 import org.apache.doris.system.Diagnoser;
 import org.apache.doris.system.SystemInfoService;
+import org.apache.doris.task.AgentClient;
+import org.apache.doris.thrift.TCheckStorageFormatResult;
 import org.apache.doris.thrift.TUnit;
 import org.apache.doris.transaction.GlobalTransactionMgr;
 import org.apache.doris.transaction.TransactionStatus;
@@ -165,7 +168,6 @@ import org.apache.doris.transaction.TransactionStatus;
 import com.google.common.base.Preconditions;
 import com.google.common.base.Strings;
 import com.google.common.collect.Lists;
-import com.google.common.collect.Maps;
 import com.google.common.collect.Sets;
 
 import org.apache.commons.lang3.tuple.Triple;
@@ -183,7 +185,6 @@ import java.util.Collections;
 import java.util.Comparator;
 import java.util.HashSet;
 import java.util.List;
-import java.util.Map;
 import java.util.Set;
 import java.util.stream.Collectors;
 
@@ -335,6 +336,8 @@ public class ShowExecutor {
             handleShowTableCreation();
         } else if (stmt instanceof ShowLastInsertStmt) {
             handleShowLastInsert();
+        } else if (stmt instanceof AdminShowTabletStorageFormatStmt) {
+            handleAdminShowTabletStorageFormat();
         } else if (stmt instanceof AdminDiagnoseTabletStmt) {
             handleAdminDiagnoseTablet();
         } else {
@@ -659,7 +662,6 @@ public class ShowExecutor {
         ShowTableStmt showTableStmt = (ShowTableStmt) stmt;
         List<List<String>> rows = Lists.newArrayList();
         Database db = ctx.getCatalog().getDbOrAnalysisException(showTableStmt.getDb());
-        Map<String, String> tableMap = Maps.newTreeMap();
         PatternMatcher matcher = null;
         if (showTableStmt.getPattern() != null) {
             matcher = PatternMatcher.createMysqlPattern(showTableStmt.getPattern(),
@@ -675,14 +677,14 @@ public class ShowExecutor {
                     PrivPredicate.SHOW)) {
                 continue;
             }
-            tableMap.put(tbl.getName(), tbl.getMysqlType());
-        }
-
-        for (Map.Entry<String, String> entry : tableMap.entrySet()) {
             if (showTableStmt.isVerbose()) {
-                rows.add(Lists.newArrayList(entry.getKey(), entry.getValue()));
+                String storageFormat = "NONE";
+                if (tbl instanceof OlapTable) {
+                    storageFormat = ((OlapTable) tbl).getStorageFormat().toString();
+                }
+                rows.add(Lists.newArrayList(tbl.getName(), tbl.getMysqlType(), storageFormat));
             } else {
-                rows.add(Lists.newArrayList(entry.getKey()));
+                rows.add(Lists.newArrayList(tbl.getName()));
             }
         }
         resultSet = new ShowResultSet(showTableStmt.getMetaData(), rows);
@@ -2132,6 +2134,43 @@ public class ShowExecutor {
         resultSet = new ShowResultSet(showMetaData, resultRowSet);
     }
 
+    private void handleAdminShowTabletStorageFormat() throws AnalysisException {
+        List<List<String>> resultRowSet = Lists.newArrayList();
+        for (Backend be : Catalog.getCurrentSystemInfo().getIdToBackend().values()) {
+            if (be.isQueryAvailable() && be.isLoadAvailable()) {
+                AgentClient client = new AgentClient(be.getHost(), be.getBePort());
+                TCheckStorageFormatResult result = client.checkStorageFormat();
+                if (result == null) {
+                    throw new AnalysisException("get tablet data from backend: " + be.getId() + "error.");
+                }
+                if (stmt.isVerbose()) {
+                    for (long tabletId : result.getV1Tablets()) {
+                        List<String> row = new ArrayList<>();
+                        row.add(String.valueOf(be.getId()));
+                        row.add(String.valueOf(tabletId));
+                        row.add("V1");
+                        resultRowSet.add(row);
+                    }
+                    for (long tabletId : result.getV2Tablets()) {
+                        List<String> row = new ArrayList<>();
+                        row.add(String.valueOf(be.getId()));
+                        row.add(String.valueOf(tabletId));
+                        row.add("V2");
+                        resultRowSet.add(row);
+                    }
+                } else {
+                    List<String> row = new ArrayList<>();
+                    row.add(String.valueOf(be.getId()));
+                    row.add(String.valueOf(result.getV1Tablets().size()));
+                    row.add(String.valueOf(result.getV2Tablets().size()));
+                    resultRowSet.add(row);
+                }
+            }
+        }
+        ShowResultSetMetaData showMetaData = stmt.getMetaData();
+        resultSet = new ShowResultSet(showMetaData, resultRowSet);
+    }
+
     private void handleAdminDiagnoseTablet() {
         AdminDiagnoseTabletStmt showStmt = (AdminDiagnoseTabletStmt) stmt;
         List<List<String>> resultRowSet = Diagnoser.diagnoseTablet(showStmt.getTabletId());
diff --git a/fe/fe-core/src/main/java/org/apache/doris/task/AgentClient.java b/fe/fe-core/src/main/java/org/apache/doris/task/AgentClient.java
index a62cb0f2f0..22dce44c3c 100644
--- a/fe/fe-core/src/main/java/org/apache/doris/task/AgentClient.java
+++ b/fe/fe-core/src/main/java/org/apache/doris/task/AgentClient.java
@@ -22,6 +22,7 @@ import org.apache.doris.common.Status;
 import org.apache.doris.thrift.BackendService;
 import org.apache.doris.thrift.TAgentResult;
 import org.apache.doris.thrift.TAgentServiceVersion;
+import org.apache.doris.thrift.TCheckStorageFormatResult;
 import org.apache.doris.thrift.TMiniLoadEtlStatusRequest;
 import org.apache.doris.thrift.TMiniLoadEtlStatusResult;
 import org.apache.doris.thrift.TMiniLoadEtlTaskRequest;
@@ -167,6 +168,21 @@ public class AgentClient {
         return result;
     }
 
+    public TCheckStorageFormatResult checkStorageFormat() {
+        TCheckStorageFormatResult result = null;
+        LOG.debug("submit make snapshot task.");
+        try {
+            borrowClient();
+            result = client.checkStorageFormat();
+            ok = true;
+        } catch (Exception e) {
+            LOG.warn("checkStorageFormat error", e);
+        } finally {
+            returnClient();
+        }
+        return result;
+    }
+
     public void deleteEtlFiles(long dbId, long jobId, String dbName, String label) {
         TDeleteEtlFilesRequest request = new TDeleteEtlFilesRequest(TAgentServiceVersion.V1, 
                 new TUniqueId(dbId, jobId), dbName, label);
@@ -197,4 +213,5 @@ public class AgentClient {
             ClientPool.backendPool.invalidateObject(address, client);
         }
     }
+
 }
diff --git a/fe/fe-core/src/test/java/org/apache/doris/analysis/ShowTableStmtTest.java b/fe/fe-core/src/test/java/org/apache/doris/analysis/ShowTableStmtTest.java
index 1e7a97312f..89c7336610 100644
--- a/fe/fe-core/src/test/java/org/apache/doris/analysis/ShowTableStmtTest.java
+++ b/fe/fe-core/src/test/java/org/apache/doris/analysis/ShowTableStmtTest.java
@@ -56,7 +56,7 @@ public class ShowTableStmtTest {
         stmt = new ShowTableStmt("abc", true, null);
         stmt.analyze(analyzer);
         Assert.assertEquals("SHOW FULL TABLES FROM testCluster:abc", stmt.toString());
-        Assert.assertEquals(2, stmt.getMetaData().getColumnCount());
+        Assert.assertEquals(3, stmt.getMetaData().getColumnCount());
         Assert.assertEquals("Tables_in_abc", stmt.getMetaData().getColumn(0).getName());
         Assert.assertEquals("Table_type", stmt.getMetaData().getColumn(1).getName());
 
@@ -64,7 +64,7 @@ public class ShowTableStmtTest {
         stmt.analyze(analyzer);
         Assert.assertEquals("bcd", stmt.getPattern());
         Assert.assertEquals("SHOW FULL TABLES FROM testCluster:abc LIKE 'bcd'", stmt.toString());
-        Assert.assertEquals(2, stmt.getMetaData().getColumnCount());
+        Assert.assertEquals(3, stmt.getMetaData().getColumnCount());
         Assert.assertEquals("Tables_in_abc", stmt.getMetaData().getColumn(0).getName());
         Assert.assertEquals("Table_type", stmt.getMetaData().getColumn(1).getName());
     }
@@ -75,4 +75,4 @@ public class ShowTableStmtTest {
         stmt.analyze(AccessTestUtil.fetchEmptyDbAnalyzer());
         Assert.fail("No exception throws");
     }
-}
\ No newline at end of file
+}
diff --git a/fe/fe-core/src/test/java/org/apache/doris/common/GenericPoolTest.java b/fe/fe-core/src/test/java/org/apache/doris/common/GenericPoolTest.java
index 3247e5e09c..feab0c256e 100644
--- a/fe/fe-core/src/test/java/org/apache/doris/common/GenericPoolTest.java
+++ b/fe/fe-core/src/test/java/org/apache/doris/common/GenericPoolTest.java
@@ -24,6 +24,7 @@ import org.apache.doris.thrift.TAgentResult;
 import org.apache.doris.thrift.TAgentTaskRequest;
 import org.apache.doris.thrift.TCancelPlanFragmentParams;
 import org.apache.doris.thrift.TCancelPlanFragmentResult;
+import org.apache.doris.thrift.TCheckStorageFormatResult;
 import org.apache.doris.thrift.TDeleteEtlFilesRequest;
 import org.apache.doris.thrift.TDiskTrashInfo;
 import org.apache.doris.thrift.TExecPlanFragmentParams;
@@ -242,6 +243,11 @@ public class GenericPoolTest {
         public void cleanTrash() throws TException {
             // TODO Auto-generated method stub
         }
+
+        @Override
+        public TCheckStorageFormatResult checkStorageFormat() throws TException {
+            return new TCheckStorageFormatResult();
+        }
     }
 
     @Test
diff --git a/fe/fe-core/src/test/java/org/apache/doris/utframe/MockedBackendFactory.java b/fe/fe-core/src/test/java/org/apache/doris/utframe/MockedBackendFactory.java
index 42dab10411..bc3e7928fd 100644
--- a/fe/fe-core/src/test/java/org/apache/doris/utframe/MockedBackendFactory.java
+++ b/fe/fe-core/src/test/java/org/apache/doris/utframe/MockedBackendFactory.java
@@ -32,6 +32,7 @@ import org.apache.doris.thrift.TBackend;
 import org.apache.doris.thrift.TBackendInfo;
 import org.apache.doris.thrift.TCancelPlanFragmentParams;
 import org.apache.doris.thrift.TCancelPlanFragmentResult;
+import org.apache.doris.thrift.TCheckStorageFormatResult;
 import org.apache.doris.thrift.TCloneReq;
 import org.apache.doris.thrift.TDeleteEtlFilesRequest;
 import org.apache.doris.thrift.TDiskTrashInfo;
@@ -319,6 +320,11 @@ public class MockedBackendFactory {
         public void cleanTrash() throws TException {
             return;
         }
+
+        @Override
+        public TCheckStorageFormatResult checkStorageFormat() throws TException {
+            return new TCheckStorageFormatResult();
+        }
     }
 
     // The default Brpc service.
diff --git a/gensrc/thrift/BackendService.thrift b/gensrc/thrift/BackendService.thrift
index 4517076f00..8e534f4677 100644
--- a/gensrc/thrift/BackendService.thrift
+++ b/gensrc/thrift/BackendService.thrift
@@ -113,6 +113,11 @@ struct TDiskTrashInfo {
     3: required i64 trash_used_capacity
 }
 
+struct TCheckStorageFormatResult {
+    1: optional list<i64> v1_tablets;
+    2: optional list<i64> v2_tablets;
+}
+
 service BackendService {
     // Called by coord to start asynchronous execution of plan fragment in backend.
     // Returns as soon as all incoming data streams have been set up.
@@ -174,4 +179,7 @@ service BackendService {
     TStreamLoadRecordResult get_stream_load_record(1: i64 last_stream_record_time);
 
     oneway void clean_trash();
+
+    // check tablet rowset type
+    TCheckStorageFormatResult check_storage_format();
 }


---------------------------------------------------------------------
To unsubscribe, e-mail: commits-unsubscribe@doris.apache.org
For additional commands, e-mail: commits-help@doris.apache.org