You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@doris.apache.org by yi...@apache.org on 2024/04/19 07:14:43 UTC

(doris) branch branch-2.1 updated: [fix](compaction)Fix single compaction to get all local versions #33849

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

yiguolei pushed a commit to branch branch-2.1
in repository https://gitbox.apache.org/repos/asf/doris.git


The following commit(s) were added to refs/heads/branch-2.1 by this push:
     new d839908214a [fix](compaction)Fix single compaction to get all local versions #33849
d839908214a is described below

commit d839908214a5139001469f02f6b173f6f1e1e9a3
Author: Sun Chenyang <cs...@gmail.com>
AuthorDate: Fri Apr 19 15:14:36 2024 +0800

    [fix](compaction)Fix single compaction to get all local versions #33849
    
    add test and comment
---
 be/src/olap/olap_server.cpp               |  2 +-
 be/src/olap/single_replica_compaction.cpp |  6 ++----
 be/src/olap/tablet.cpp                    | 11 +++++-----
 be/src/olap/tablet.h                      |  4 +++-
 be/test/olap/tablet_test.cpp              | 36 +++++++++++++++++++++++++++++++
 5 files changed, 48 insertions(+), 11 deletions(-)

diff --git a/be/src/olap/olap_server.cpp b/be/src/olap/olap_server.cpp
index a2b5bf56de9..ce805b842dd 100644
--- a/be/src/olap/olap_server.cpp
+++ b/be/src/olap/olap_server.cpp
@@ -800,7 +800,7 @@ void StorageEngine::get_tablet_rowset_versions(const PGetTabletVersionsRequest*
         response->mutable_status()->set_status_code(TStatusCode::CANCELLED);
         return;
     }
-    std::vector<Version> local_versions = tablet->get_all_versions();
+    std::vector<Version> local_versions = tablet->get_all_local_versions();
     for (const auto& local_version : local_versions) {
         auto version = response->add_versions();
         version->set_first(local_version.first);
diff --git a/be/src/olap/single_replica_compaction.cpp b/be/src/olap/single_replica_compaction.cpp
index 038c3893497..9f0183a942c 100644
--- a/be/src/olap/single_replica_compaction.cpp
+++ b/be/src/olap/single_replica_compaction.cpp
@@ -217,10 +217,8 @@ Status SingleReplicaCompaction::_get_rowset_verisons_from_peer(
 
 bool SingleReplicaCompaction::_find_rowset_to_fetch(const std::vector<Version>& peer_versions,
                                                     Version* proper_version) {
-    //  get local versions
-    std::vector<Version> local_versions = _tablet->get_all_versions();
-    std::sort(local_versions.begin(), local_versions.end(),
-              [](const Version& left, const Version& right) { return left.first < right.first; });
+    //  already sorted
+    std::vector<Version> local_versions = tablet()->get_all_local_versions();
     for (const auto& v : local_versions) {
         VLOG_CRITICAL << _tablet->tablet_id() << " tablet local version: " << v.first << " - "
                       << v.second;
diff --git a/be/src/olap/tablet.cpp b/be/src/olap/tablet.cpp
index e6f421d5ebe..22dac0a6312 100644
--- a/be/src/olap/tablet.cpp
+++ b/be/src/olap/tablet.cpp
@@ -1895,13 +1895,14 @@ void Tablet::execute_single_replica_compaction(SingleReplicaCompaction& compacti
     set_last_failure_time(this, compaction, 0);
 }
 
-std::vector<Version> Tablet::get_all_versions() {
+std::vector<Version> Tablet::get_all_local_versions() {
     std::vector<Version> local_versions;
     {
-        std::lock_guard<std::shared_mutex> wrlock(_meta_lock);
-        SCOPED_SIMPLE_TRACE_IF_TIMEOUT(TRACE_TABLET_LOCK_THRESHOLD);
-        for (const auto& it : _rs_version_map) {
-            local_versions.emplace_back(it.first);
+        std::shared_lock rlock(_meta_lock);
+        for (const auto& [version, rs] : _rs_version_map) {
+            if (rs->is_local()) {
+                local_versions.emplace_back(version);
+            }
         }
     }
     std::sort(local_versions.begin(), local_versions.end(),
diff --git a/be/src/olap/tablet.h b/be/src/olap/tablet.h
index 8b28b57897d..95c70196354 100644
--- a/be/src/olap/tablet.h
+++ b/be/src/olap/tablet.h
@@ -287,7 +287,9 @@ public:
             const std::set<int32_t>& alter_index_uids, bool is_drop_op);
 
     std::vector<RowsetSharedPtr> pick_candidate_rowsets_to_single_replica_compaction();
-    std::vector<Version> get_all_versions();
+    // used for single compaction to get the local versions
+    // Single compaction does not require remote rowsets and cannot violate the cooldown semantics
+    std::vector<Version> get_all_local_versions();
 
     std::vector<RowsetSharedPtr> pick_first_consecutive_empty_rowsets(int limit);
 
diff --git a/be/test/olap/tablet_test.cpp b/be/test/olap/tablet_test.cpp
index f5778d47982..8d84b5141c3 100644
--- a/be/test/olap/tablet_test.cpp
+++ b/be/test/olap/tablet_test.cpp
@@ -148,6 +148,20 @@ public:
         pb1->set_tablet_schema(_tablet_meta->tablet_schema());
     }
 
+    void init_rs_meta_resource(RowsetMetaSharedPtr& pb1, int64_t start, int64_t end,
+                               bool is_local) {
+        RowsetMetaPB rowset_meta_pb;
+        json2pb::JsonToProtoMessage(_json_rowset_meta, &rowset_meta_pb);
+        rowset_meta_pb.set_start_version(start);
+        rowset_meta_pb.set_end_version(end);
+        rowset_meta_pb.set_creation_time(10000);
+        if (!is_local) {
+            rowset_meta_pb.set_resource_id("100");
+        }
+        pb1->init_from_pb(rowset_meta_pb);
+        pb1->set_tablet_schema(_tablet_meta->tablet_schema());
+    }
+
     void init_all_rs_meta(std::vector<RowsetMetaSharedPtr>* rs_metas) {
         RowsetMetaSharedPtr ptr1(new RowsetMeta());
         init_rs_meta(ptr1, 0, 0);
@@ -398,4 +412,26 @@ TEST_F(TestTablet, cooldown_policy) {
     }
 }
 
+TEST_F(TestTablet, get_local_versions) {
+    // 10 remote rowsets
+    for (int i = 1; i <= 10; i++) {
+        auto ptr = std::make_shared<RowsetMeta>();
+        init_rs_meta_resource(ptr, i, i, false);
+        static_cast<void>(_tablet_meta->add_rs_meta(ptr));
+    }
+
+    // 20 local rowsets
+    for (int i = 11; i <= 30; i++) {
+        auto ptr = std::make_shared<RowsetMeta>();
+        init_rs_meta_resource(ptr, i, i, true);
+        static_cast<void>(_tablet_meta->add_rs_meta(ptr));
+    }
+
+    static_cast<void>(_data_dir->init());
+    TabletSharedPtr _tablet(new Tablet(*k_engine, _tablet_meta, _data_dir.get()));
+    static_cast<void>(_tablet->init());
+    const auto& local_versions = _tablet->get_all_local_versions();
+    ASSERT_EQ(local_versions.size(), 20);
+}
+
 } // namespace doris


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