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/12/29 04:37:32 UTC

[doris] 01/05: [feature](BE)pad missed version with empty rowset (#15030)

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

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

commit 740363019c36bb1bb76a9c7b2ad2ce193ae0b66e
Author: AlexYue <yj...@qq.com>
AuthorDate: Thu Dec 29 11:20:44 2022 +0800

    [feature](BE)pad missed version with empty rowset (#15030)
    
    If all replicas of one tablet are broken, user can use this http api to pad the missed version with empty rowset.
---
 be/src/http/CMakeLists.txt                         |   1 +
 be/src/http/action/pad_rowset_action.cpp           | 105 +++++++++++++++++++++
 be/src/http/action/pad_rowset_action.h             |  44 +++++++++
 be/src/service/http_service.cpp                    |   4 +
 be/test/olap/tablet_test.cpp                       |  55 +++++++++++
 .../docs/admin-manual/http-actions/pad-rowset.md   |  41 ++++++++
 docs/sidebars.json                                 |   1 +
 .../docs/admin-manual/http-actions/pad_rowset.md   |  43 +++++++++
 8 files changed, 294 insertions(+)

diff --git a/be/src/http/CMakeLists.txt b/be/src/http/CMakeLists.txt
index a23d679391..e38ff4d1ee 100644
--- a/be/src/http/CMakeLists.txt
+++ b/be/src/http/CMakeLists.txt
@@ -35,6 +35,7 @@ add_library(Webserver STATIC
   http_client.cpp
   action/download_action.cpp
   action/monitor_action.cpp
+  action/pad_rowset_action.cpp
   action/health_action.cpp
   action/tablet_migration_action.cpp
   action/tablets_info_action.cpp
diff --git a/be/src/http/action/pad_rowset_action.cpp b/be/src/http/action/pad_rowset_action.cpp
new file mode 100644
index 0000000000..df2721f50b
--- /dev/null
+++ b/be/src/http/action/pad_rowset_action.cpp
@@ -0,0 +1,105 @@
+// 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 "http/action/pad_rowset_action.h"
+
+#include <memory>
+#include <mutex>
+
+#include "http/http_channel.h"
+#include "olap/olap_common.h"
+#include "olap/rowset/beta_rowset_writer.h"
+#include "olap/rowset/rowset.h"
+#include "olap/storage_engine.h"
+
+namespace doris {
+
+const std::string TABLET_ID = "tablet_id";
+const std::string START_VERSION = "start_version";
+const std::string END_VERSION = "end_version";
+
+Status check_one_param(const std::string& param_val, const std::string& param_name) {
+    if (param_val.empty()) {
+        return Status::InternalError("paramater {} not specified in url", param_name);
+    }
+    return Status::OK();
+}
+
+void PadRowsetAction::handle(HttpRequest* req) {
+    LOG(INFO) << "accept one request " << req->debug_string();
+    Status status = _handle(req);
+    std::string result = status.to_json();
+    LOG(INFO) << "handle request result:" << result;
+    if (status.ok()) {
+        HttpChannel::send_reply(req, HttpStatus::OK, result);
+    } else {
+        HttpChannel::send_reply(req, HttpStatus::INTERNAL_SERVER_ERROR, result);
+    }
+}
+
+Status PadRowsetAction::check_param(HttpRequest* req) {
+    RETURN_IF_ERROR(check_one_param(req->param(TABLET_ID), TABLET_ID));
+    RETURN_IF_ERROR(check_one_param(req->param(START_VERSION), START_VERSION));
+    RETURN_IF_ERROR(check_one_param(req->param(END_VERSION), END_VERSION));
+    return Status::OK();
+}
+
+Status PadRowsetAction::_handle(HttpRequest* req) {
+    RETURN_IF_ERROR(check_param(req));
+
+    const std::string& tablet_id_str = req->param(TABLET_ID);
+    const std::string& start_version_str = req->param(START_VERSION);
+    const std::string& end_version_str = req->param(END_VERSION);
+
+    // valid str format
+    int64_t tablet_id = std::atol(tablet_id_str.c_str());
+    int32_t start_version = std::atoi(start_version_str.c_str());
+    int32_t end_version = std::atoi(end_version_str.c_str());
+    if (start_version < 0 || end_version < 0 || end_version < start_version) {
+        return Status::InternalError("Invalid input version");
+    }
+
+    auto tablet = StorageEngine::instance()->tablet_manager()->get_tablet(tablet_id);
+    if (nullptr == tablet) {
+        return Status::InternalError("Unknown tablet id {}", tablet_id);
+    }
+    return _pad_rowset(tablet, Version(start_version, end_version));
+}
+
+Status PadRowsetAction::_pad_rowset(TabletSharedPtr tablet, const Version& version) {
+    if (tablet->check_version_exist(version)) {
+        return Status::InternalError("Input version {} exists", version.to_string());
+    }
+
+    std::unique_ptr<RowsetWriter> writer;
+    RETURN_IF_ERROR(tablet->create_rowset_writer(version, VISIBLE, NONOVERLAPPING,
+                                                 tablet->tablet_schema(), -1, -1, &writer));
+    auto rowset = writer->build();
+    rowset->make_visible(version);
+
+    std::vector<RowsetSharedPtr> to_add {rowset};
+    std::vector<RowsetSharedPtr> to_delete;
+    {
+        std::unique_lock wlock(tablet->get_header_lock());
+        tablet->modify_rowsets(to_add, to_delete);
+        tablet->save_meta();
+    }
+
+    return Status::OK();
+}
+
+} // namespace doris
diff --git a/be/src/http/action/pad_rowset_action.h b/be/src/http/action/pad_rowset_action.h
new file mode 100644
index 0000000000..f6036dc9fa
--- /dev/null
+++ b/be/src/http/action/pad_rowset_action.h
@@ -0,0 +1,44 @@
+// 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.
+
+#pragma once
+
+#include "common/status.h"
+#include "http/http_handler.h"
+#include "http/http_request.h"
+#include "olap/tablet.h"
+
+namespace doris {
+
+class PadRowsetAction : public HttpHandler {
+public:
+    PadRowsetAction() = default;
+
+    ~PadRowsetAction() override = default;
+
+    void handle(HttpRequest* req) override;
+
+private:
+    Status _handle(HttpRequest* req);
+    Status check_param(HttpRequest* req);
+
+#ifdef BE_TEST
+public:
+#endif
+    Status _pad_rowset(TabletSharedPtr tablet, const Version& version);
+};
+} // end namespace doris
\ No newline at end of file
diff --git a/be/src/service/http_service.cpp b/be/src/service/http_service.cpp
index b62e54e6b1..06334bde29 100644
--- a/be/src/service/http_service.cpp
+++ b/be/src/service/http_service.cpp
@@ -26,6 +26,7 @@
 #include "http/action/health_action.h"
 #include "http/action/meta_action.h"
 #include "http/action/metrics_action.h"
+#include "http/action/pad_rowset_action.h"
 #include "http/action/pprof_actions.h"
 #include "http/action/reload_tablet_action.h"
 #include "http/action/reset_rpc_channel_action.h"
@@ -180,6 +181,9 @@ Status HttpService::start() {
     _ev_http_server->register_handler(HttpMethod::POST, "/api/check_tablet_segment_lost",
                                       check_tablet_segment_action);
 
+    PadRowsetAction* pad_rowset_action = _pool.add(new PadRowsetAction());
+    _ev_http_server->register_handler(HttpMethod::POST, "api/pad_rowset", pad_rowset_action);
+
     _ev_http_server->start();
     return Status::OK();
 }
diff --git a/be/test/olap/tablet_test.cpp b/be/test/olap/tablet_test.cpp
index 6e9990dd3b..b29443776b 100644
--- a/be/test/olap/tablet_test.cpp
+++ b/be/test/olap/tablet_test.cpp
@@ -21,6 +21,7 @@
 
 #include <sstream>
 
+#include "http/action/pad_rowset_action.h"
 #include "olap/olap_define.h"
 #include "olap/rowset/beta_rowset.h"
 #include "olap/storage_engine.h"
@@ -28,6 +29,7 @@
 #include "olap/tablet_meta.h"
 #include "olap/tablet_schema_cache.h"
 #include "testutil/mock_rowset.h"
+#include "util/file_utils.h"
 #include "util/time.h"
 
 using namespace std;
@@ -37,6 +39,8 @@ namespace doris {
 using RowsetMetaSharedContainerPtr = std::shared_ptr<std::vector<RowsetMetaSharedPtr>>;
 
 static StorageEngine* k_engine = nullptr;
+static const std::string kTestDir = "/data_test/data/tablet_test";
+static const uint32_t MAX_PATH_LEN = 1024;
 
 class TestTablet : public testing::Test {
 public:
@@ -92,6 +96,17 @@ public:
                 }]
             }
         })";
+        char buffer[MAX_PATH_LEN];
+        EXPECT_NE(getcwd(buffer, MAX_PATH_LEN), nullptr);
+        absolute_dir = std::string(buffer) + kTestDir;
+
+        if (FileUtils::check_exist(absolute_dir)) {
+            EXPECT_TRUE(FileUtils::remove_all(absolute_dir).ok());
+        }
+        EXPECT_TRUE(FileUtils::create_dir(absolute_dir).ok());
+        EXPECT_TRUE(FileUtils::create_dir(absolute_dir + "/tablet_path").ok());
+        _data_dir = std::make_unique<DataDir>(absolute_dir);
+        _data_dir->update_capacity();
 
         doris::EngineOptions options;
         k_engine = new StorageEngine(options);
@@ -99,6 +114,9 @@ public:
     }
 
     void TearDown() override {
+        if (FileUtils::check_exist(absolute_dir)) {
+            EXPECT_TRUE(FileUtils::remove_all(absolute_dir).ok());
+        }
         if (k_engine != nullptr) {
             k_engine->stop();
             delete k_engine;
@@ -224,6 +242,8 @@ public:
 protected:
     std::string _json_rowset_meta;
     TabletMetaSharedPtr _tablet_meta;
+    string absolute_dir;
+    std::unique_ptr<DataDir> _data_dir;
 };
 
 TEST_F(TestTablet, delete_expired_stale_rowset) {
@@ -252,6 +272,41 @@ TEST_F(TestTablet, delete_expired_stale_rowset) {
     _tablet.reset();
 }
 
+TEST_F(TestTablet, pad_rowset) {
+    std::vector<RowsetMetaSharedPtr> rs_metas;
+    auto ptr1 = std::make_shared<RowsetMeta>();
+    init_rs_meta(ptr1, 1, 2);
+    rs_metas.push_back(ptr1);
+    RowsetSharedPtr rowset1 = make_shared<BetaRowset>(nullptr, "", ptr1);
+
+    auto ptr2 = std::make_shared<RowsetMeta>();
+    init_rs_meta(ptr2, 3, 4);
+    rs_metas.push_back(ptr2);
+    RowsetSharedPtr rowset2 = make_shared<BetaRowset>(nullptr, "", ptr2);
+
+    auto ptr3 = std::make_shared<RowsetMeta>();
+    init_rs_meta(ptr3, 6, 7);
+    rs_metas.push_back(ptr3);
+    RowsetSharedPtr rowset3 = make_shared<BetaRowset>(nullptr, "", ptr3);
+
+    for (auto& rowset : rs_metas) {
+        _tablet_meta->add_rs_meta(rowset);
+    }
+
+    _data_dir->init();
+    TabletSharedPtr _tablet(new Tablet(_tablet_meta, _data_dir.get()));
+    _tablet->init();
+
+    Version version(5, 5);
+    std::vector<RowsetReaderSharedPtr> readers;
+    ASSERT_FALSE(_tablet->capture_rs_readers(version, &readers).ok());
+    readers.clear();
+
+    PadRowsetAction action;
+    action._pad_rowset(_tablet, version);
+    ASSERT_TRUE(_tablet->capture_rs_readers(version, &readers).ok());
+}
+
 TEST_F(TestTablet, cooldown_policy) {
     std::vector<RowsetMetaSharedPtr> rs_metas;
     RowsetMetaSharedPtr ptr1(new RowsetMeta());
diff --git a/docs/en/docs/admin-manual/http-actions/pad-rowset.md b/docs/en/docs/admin-manual/http-actions/pad-rowset.md
new file mode 100644
index 0000000000..9ff6b89053
--- /dev/null
+++ b/docs/en/docs/admin-manual/http-actions/pad-rowset.md
@@ -0,0 +1,41 @@
+---
+{
+    "title": "PAD ROWSET",
+    "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.
+-->
+
+# PAD ROWSET
+## description
+   
+    Pad one empty rowset as one substitute for error replica.
+
+    METHOD: POST
+    URI: http://be_host:be_http_port/api/pad_rowset?tablet_id=xxx&start_version=xxx&end_version=xxx
+
+## example
+
+    curl -X POST "http://hostname:8088/api/pad_rowset?tablet_id=123456\&start_version=1111111\$end_version=1111112"
+
+## keyword
+
+    ROWSET,TABLET,ROWSET,TABLET
diff --git a/docs/sidebars.json b/docs/sidebars.json
index 71dd56628d..0dfcd8a982 100644
--- a/docs/sidebars.json
+++ b/docs/sidebars.json
@@ -1105,6 +1105,7 @@
                             ]
                         },
                         "admin-manual/http-actions/restore-tablet",
+                        "admin-manual/http-actions/pad-rowset",
                         "admin-manual/http-actions/get-load-state",
                         "admin-manual/http-actions/tablet-migration-action",
                         "admin-manual/http-actions/cancel-label",
diff --git a/docs/zh-CN/docs/admin-manual/http-actions/pad_rowset.md b/docs/zh-CN/docs/admin-manual/http-actions/pad_rowset.md
new file mode 100644
index 0000000000..dba241fca7
--- /dev/null
+++ b/docs/zh-CN/docs/admin-manual/http-actions/pad_rowset.md
@@ -0,0 +1,43 @@
+---
+{
+    "title": "PAD ROWSET",
+    "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.
+-->
+
+# PAD ROWSET
+## description
+   
+    该功能用于使用一个空的rowset填充损坏的副本。
+
+    说明:这个功能暂时只在be服务中提供一个http接口。如果要使用,
+    需要向要进行数据恢复的那台be机器的http端口发送pad rowset api请求。api格式如下:
+    METHOD: POST
+    URI: http://be_host:be_http_port/api/pad_rowset?tablet_id=xxx&start_version=xxx&end_version=xxx
+
+## example
+
+    curl -X POST "http://hostname:8088/api/pad_rowset?tablet_id=123456\&start_version=1111111\&end_version=1111112"
+
+## keyword
+
+    PAD,ROWSET,PAD,ROWSET


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