You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@doris.apache.org by zh...@apache.org on 2020/06/06 03:36:38 UTC

[incubator-doris] branch master updated: [Bug] Fix a bug that tablet's _preferred_rowset_type may be modified to BETA_ROWSET after cloned (#3750)

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

zhaoc 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 3b6a781  [Bug] Fix a bug that tablet's _preferred_rowset_type may be modified to BETA_ROWSET after cloned (#3750)
3b6a781 is described below

commit 3b6a781862abb5475c226da79b431510171c4756
Author: Yingchun Lai <40...@qq.com>
AuthorDate: Sat Jun 6 11:36:28 2020 +0800

    [Bug] Fix a bug that tablet's _preferred_rowset_type may be modified to BETA_ROWSET after cloned (#3750)
    
    TabletMeta's _preferred_rowset_type is not initialized after object constructing and
    may be a random value, and this field is not updated when create ALPHA_ROWSET tablet,
    and it will not be serialized into pb in this case. So if cloning an ALPHA_ROWSET
    tablet from another BE, this new created local tablet's _preferred_rowset_type field
    may be random as BETA_ROWSET and can not be overwrote after cloned, then new input
    rows will be wrote as BETA_ROWSET format which is not we expect.
    This patch fix this bug by giving _preferred_rowset_type a default value and updating
    this field when create any type of tablet, and add an unit test and related overwrite
    equal operator functions.
---
 be/src/olap/rowset/rowset_meta.h  | 12 +++++++
 be/src/olap/tablet_manager.cpp    |  2 ++
 be/src/olap/tablet_meta.cpp       | 42 ++++++++++++++++++++++++
 be/src/olap/tablet_meta.h         | 24 +++++++++++---
 be/src/olap/tablet_schema.cpp     | 67 +++++++++++++++++++++++++++++++++++----
 be/src/olap/tablet_schema.h       | 32 +++++++++++++------
 be/src/util/uid_util.h            |  4 +--
 be/test/olap/CMakeLists.txt       |  1 +
 be/test/olap/tablet_meta_test.cpp | 48 ++++++++++++++++++++++++++++
 9 files changed, 210 insertions(+), 22 deletions(-)

diff --git a/be/src/olap/rowset/rowset_meta.h b/be/src/olap/rowset/rowset_meta.h
index a42b233..277deab 100644
--- a/be/src/olap/rowset/rowset_meta.h
+++ b/be/src/olap/rowset/rowset_meta.h
@@ -25,6 +25,7 @@
 #include <vector>
 
 #include "olap/olap_common.h"
+#include "google/protobuf/util/message_differencer.h"
 #include "json2pb/json_to_pb.h"
 #include "json2pb/pb_to_json.h"
 #include "common/logging.h"
@@ -392,6 +393,17 @@ private:
         }
     }
 
+    friend bool operator==(const RowsetMeta& a, const RowsetMeta& b) {
+        if (a._rowset_id != b._rowset_id) return false;
+        if (a._is_removed_from_rowset_meta != b._is_removed_from_rowset_meta) return false;
+        if (!google::protobuf::util::MessageDifferencer::Equals(a._rowset_meta_pb, b._rowset_meta_pb)) return false;
+        return true;
+    }
+
+    friend bool operator!=(const RowsetMeta& a, const RowsetMeta& b) {
+        return !(a == b);
+    }
+
 private:
     RowsetMetaPB _rowset_meta_pb;
     RowsetId _rowset_id;
diff --git a/be/src/olap/tablet_manager.cpp b/be/src/olap/tablet_manager.cpp
index 78bd2f6..3c3fbe5 100644
--- a/be/src/olap/tablet_manager.cpp
+++ b/be/src/olap/tablet_manager.cpp
@@ -1307,6 +1307,8 @@ OLAPStatus TabletManager::_create_tablet_meta_unlocked(const TCreateTabletReq& r
     // TODO(lingbin): when beta-rowset is default, should remove it
     if (request.__isset.storage_format && request.storage_format == TStorageFormat::V2) {
         (*tablet_meta)->set_preferred_rowset_type(BETA_ROWSET);
+    } else {
+        (*tablet_meta)->set_preferred_rowset_type(ALPHA_ROWSET);
     }
     return res;
 }
diff --git a/be/src/olap/tablet_meta.cpp b/be/src/olap/tablet_meta.cpp
index 2149629..4b9a45f 100755
--- a/be/src/olap/tablet_meta.cpp
+++ b/be/src/olap/tablet_meta.cpp
@@ -669,4 +669,46 @@ OLAPStatus TabletMeta::set_partition_id(int64_t partition_id) {
     return OLAP_SUCCESS;
 }
 
+bool operator==(const AlterTabletTask& a, const AlterTabletTask& b) {
+    if (a._alter_state != b._alter_state) return false;
+    if (a._related_tablet_id != b._related_tablet_id) return false;
+    if (a._related_schema_hash != b._related_schema_hash) return false;
+    if (a._alter_type != b._alter_type) return false;
+    return true;
+}
+
+bool operator!=(const AlterTabletTask& a, const AlterTabletTask& b) {
+    return !(a == b);
+}
+
+bool operator==(const TabletMeta& a, const TabletMeta& b) {
+    if (a._table_id != b._table_id) return false;
+    if (a._partition_id != b._partition_id) return false;
+    if (a._tablet_id != b._tablet_id) return false;
+    if (a._schema_hash != b._schema_hash) return false;
+    if (a._shard_id != b._shard_id) return false;
+    if (a._creation_time != b._creation_time) return false;
+    if (a._cumulative_layer_point != b._cumulative_layer_point) return false;
+    if (a._tablet_uid != b._tablet_uid) return false;
+    if (a._tablet_type != b._tablet_type) return false;
+    if (a._tablet_state != b._tablet_state) return false;
+    if (a._schema != b._schema) return false;
+    if (a._rs_metas.size() != b._rs_metas.size()) return false;
+    for (int i = 0; i < a._rs_metas.size(); ++i) {
+        if (a._rs_metas[i] != b._rs_metas[i]) return false;
+    }
+    if (a._inc_rs_metas.size() != b._inc_rs_metas.size()) return false;
+    for (int i = 0; i < a._inc_rs_metas.size(); ++i) {
+        if (a._inc_rs_metas[i] != b._inc_rs_metas[i]) return false;
+    }
+    if (a._alter_task != b._alter_task) return false;
+    if (a._in_restore_mode != b._in_restore_mode) return false;
+    if (a._preferred_rowset_type != b._preferred_rowset_type) return false;
+    return true;
+}
+
+bool operator!=(const TabletMeta& a, const TabletMeta& b) {
+    return !(a == b);
+}
+
 }  // namespace doris
diff --git a/be/src/olap/tablet_meta.h b/be/src/olap/tablet_meta.h
index 4623520..f2c51cb 100644
--- a/be/src/olap/tablet_meta.h
+++ b/be/src/olap/tablet_meta.h
@@ -85,13 +85,19 @@ public:
     inline const AlterTabletType& alter_type() const { return _alter_type; }
     inline void set_alter_type(AlterTabletType alter_type) { _alter_type = alter_type; }
 
+    friend bool operator==(const AlterTabletTask& a, const AlterTabletTask& b);
+    friend bool operator!=(const AlterTabletTask& a, const AlterTabletTask& b);
+
 private:
-    AlterTabletState _alter_state;
-    int64_t _related_tablet_id;
-    int32_t _related_schema_hash;
-    AlterTabletType _alter_type;
+    AlterTabletState _alter_state = ALTER_PREPARED;
+    int64_t _related_tablet_id = 0;
+    int32_t _related_schema_hash = 0;
+    AlterTabletType _alter_type = SCHEMA_CHANGE;
 };
 
+bool operator==(const AlterTabletTask& a, const AlterTabletTask& b);
+bool operator!=(const AlterTabletTask& a, const AlterTabletTask& b);
+
 typedef std::shared_ptr<AlterTabletTask> AlterTabletTaskSharedPtr;
 
 // Class encapsulates meta of tablet.
@@ -196,6 +202,10 @@ public:
 private:
     OLAPStatus _save_meta(DataDir* data_dir);
 
+    // _del_pred_array is ignored to compare.
+    friend bool operator==(const TabletMeta& a, const TabletMeta& b);
+    friend bool operator!=(const TabletMeta& a, const TabletMeta& b);
+
 private:
     int64_t _table_id = 0;
     int64_t _partition_id = 0;
@@ -214,7 +224,7 @@ private:
     DelPredicateArray _del_pred_array;
     AlterTabletTaskSharedPtr _alter_task;
     bool _in_restore_mode = false;
-    RowsetTypePB _preferred_rowset_type;
+    RowsetTypePB _preferred_rowset_type = ALPHA_ROWSET;
 
     RWMutex _meta_lock;
 };
@@ -315,6 +325,10 @@ inline const std::vector<RowsetMetaSharedPtr>& TabletMeta::all_inc_rs_metas() co
     return _inc_rs_metas;
 }
 
+// Only for unit test now.
+bool operator==(const TabletMeta& a, const TabletMeta& b);
+bool operator!=(const TabletMeta& a, const TabletMeta& b);
+
 }  // namespace doris
 
 #endif // DORIS_BE_SRC_OLAP_OLAP_TABLET_META_H
diff --git a/be/src/olap/tablet_schema.cpp b/be/src/olap/tablet_schema.cpp
index ac20d57..7f56982 100644
--- a/be/src/olap/tablet_schema.cpp
+++ b/be/src/olap/tablet_schema.cpp
@@ -337,14 +337,12 @@ void TabletColumn::to_schema_pb(ColumnPB* column) {
     }
 }
 
-TabletSchema::TabletSchema()
-    : _num_columns(0),
-      _num_key_columns(0),
-      _num_null_columns(0),
-      _num_short_key_columns(0) { }
-
 void TabletSchema::init_from_pb(const TabletSchemaPB& schema) {
     _keys_type = schema.keys_type();
+    _num_columns = 0;
+    _num_key_columns = 0;
+    _num_null_columns = 0;
+    _cols.clear();
     for (auto& column_pb : schema.column()) {
         TabletColumn column;
         column.init_from_pb(column_pb);
@@ -420,4 +418,61 @@ const TabletColumn& TabletSchema::column(size_t ordinal) const {
     return _cols[ordinal];
 }
 
+bool operator==(const TabletColumn& a, const TabletColumn& b) {
+    if (a._unique_id != b._unique_id) return false;
+    if (a._col_name != b._col_name) return false;
+    if (a._type != b._type) return false;
+    if (a._is_key != b._is_key) return false;
+    if (a._aggregation != b._aggregation) return false;
+    if (a._is_nullable != b._is_nullable) return false;
+    if (a._has_default_value != b._has_default_value) return false;
+    if (a._has_default_value) {
+        if (a._default_value != b._default_value) return false;
+    }
+    if (a._is_decimal != b._is_decimal) return false;
+    if (a._is_decimal) {
+        if (a._precision != b._precision) return false;
+        if (a._frac != b._frac) return false;
+    }
+    if (a._length != b._length) return false;
+    if (a._index_length != b._index_length) return false;
+    if (a._is_bf_column != b._is_bf_column) return false;
+    if (a._has_referenced_column != b._has_referenced_column) return false;
+    if (a._has_referenced_column) {
+        if (a._referenced_column_id != b._referenced_column_id) return false;
+        if (a._referenced_column != b._referenced_column) return false;
+    }
+    if (a._has_bitmap_index != b._has_bitmap_index) return false;
+    return true;
+}
+
+bool operator!=(const TabletColumn& a, const TabletColumn& b) {
+    return !(a == b);
+}
+
+bool operator==(const TabletSchema& a, const TabletSchema& b) {
+    if (a._keys_type != b._keys_type) return false;
+    if (a._cols.size() != b._cols.size()) return false;
+    for (int i = 0; i < a._cols.size(); ++i) {
+        if (a._cols[i] != b._cols[i]) return false;
+    }
+    if (a._num_columns != b._num_columns) return false;
+    if (a._num_key_columns != b._num_key_columns) return false;
+    if (a._num_null_columns != b._num_null_columns) return false;
+    if (a._num_short_key_columns != b._num_short_key_columns) return false;
+    if (a._num_rows_per_row_block != b._num_rows_per_row_block) return false;
+    if (a._compress_kind != b._compress_kind) return false;
+    if (a._next_column_unique_id != b._next_column_unique_id) return false;
+    if (a._has_bf_fpp != b._has_bf_fpp) return false;
+    if (a._has_bf_fpp) {
+        if (std::abs(a._bf_fpp - b._bf_fpp) > 1e-6) return false;
+    }
+    if (a._is_in_memory != b._is_in_memory) return false;
+    return true;
+}
+
+bool operator!=(const TabletSchema& a, const TabletSchema& b) {
+  return !(a == b);
+}
+
 } // namespace doris
diff --git a/be/src/olap/tablet_schema.h b/be/src/olap/tablet_schema.h
index 4dbfae5..33cf4e8 100644
--- a/be/src/olap/tablet_schema.h
+++ b/be/src/olap/tablet_schema.h
@@ -52,6 +52,9 @@ public:
     int precision() const { return _precision; }
     int frac() const { return _frac; }
 
+    friend bool operator==(const TabletColumn& a, const TabletColumn& b);
+    friend bool operator!=(const TabletColumn& a, const TabletColumn& b);
+
     static std::string get_string_by_field_type(FieldType type);
     static std::string get_string_by_aggregation_type(FieldAggregationMethod aggregation_type);
     static FieldType get_field_type_by_string(const std::string& str);
@@ -85,9 +88,12 @@ private:
     bool _has_bitmap_index = false;
 };
 
+bool operator==(const TabletColumn& a, const TabletColumn& b);
+bool operator!=(const TabletColumn& a, const TabletColumn& b);
+
 class TabletSchema {
 public:
-    TabletSchema();
+    TabletSchema() = default;
     void init_from_pb(const TabletSchemaPB& schema);
     void to_schema_pb(TabletSchemaPB* tablet_meta_pb);
     size_t row_size() const;
@@ -107,22 +113,30 @@ public:
     inline void set_is_in_memory (bool is_in_memory) {
         _is_in_memory = is_in_memory;
     }
+
 private:
-    KeysType _keys_type;
+    friend bool operator==(const TabletSchema& a, const TabletSchema& b);
+    friend bool operator!=(const TabletSchema& a, const TabletSchema& b);
+
+private:
+    KeysType _keys_type = DUP_KEYS;
     std::vector<TabletColumn> _cols;
-    size_t _num_columns;
-    size_t _num_key_columns;
-    size_t _num_null_columns;
-    size_t _num_short_key_columns;
-    size_t _num_rows_per_row_block;
-    CompressKind _compress_kind;
-    size_t _next_column_unique_id;
+    size_t _num_columns = 0;
+    size_t _num_key_columns = 0;
+    size_t _num_null_columns = 0;
+    size_t _num_short_key_columns = 0;
+    size_t _num_rows_per_row_block = 0;
+    CompressKind _compress_kind = COMPRESS_NONE;
+    size_t _next_column_unique_id = 0;
 
     bool _has_bf_fpp = false;
     double _bf_fpp = 0;
     bool _is_in_memory = false;
 };
 
+bool operator==(const TabletSchema& a, const TabletSchema& b);
+bool operator!=(const TabletSchema& a, const TabletSchema& b);
+
 } // namespace doris
 
 #endif // DORIS_BE_SRC_OLAP_TABLET_SCHEMA_H
diff --git a/be/src/util/uid_util.h b/be/src/util/uid_util.h
index eb4019f..dc2d938 100644
--- a/be/src/util/uid_util.h
+++ b/be/src/util/uid_util.h
@@ -58,8 +58,8 @@ inline void from_hex(T* ret, const std::string& buf) {
 }
 
 struct UniqueId {
-    int64_t hi;
-    int64_t lo;
+    int64_t hi = 0;
+    int64_t lo = 0;
 
     UniqueId(int64_t hi_, int64_t lo_) : hi(hi_), lo(lo_) { }
     UniqueId(const TUniqueId& tuid) : hi(tuid.hi), lo(tuid.lo) { }
diff --git a/be/test/olap/CMakeLists.txt b/be/test/olap/CMakeLists.txt
index fc9f2a8..2a2e798 100644
--- a/be/test/olap/CMakeLists.txt
+++ b/be/test/olap/CMakeLists.txt
@@ -63,6 +63,7 @@ ADD_BE_TEST(rowset/segment_v2/frame_of_reference_page_test)
 ADD_BE_TEST(rowset/segment_v2/block_bloom_filter_test)
 ADD_BE_TEST(rowset/segment_v2/bloom_filter_index_reader_writer_test)
 ADD_BE_TEST(rowset/segment_v2/zone_map_index_test)
+ADD_BE_TEST(tablet_meta_test)
 ADD_BE_TEST(tablet_meta_manager_test)
 ADD_BE_TEST(tablet_mgr_test)
 ADD_BE_TEST(rowset/rowset_meta_manager_test)
diff --git a/be/test/olap/tablet_meta_test.cpp b/be/test/olap/tablet_meta_test.cpp
new file mode 100755
index 0000000..9a054b4
--- /dev/null
+++ b/be/test/olap/tablet_meta_test.cpp
@@ -0,0 +1,48 @@
+// 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 <string>
+
+#include <gtest/gtest.h>
+
+#include "olap/tablet_meta.h"
+
+namespace doris {
+
+TEST(TabletMetaTest, SaveAndParse) {
+    std::string meta_path = "./be/test/olap/test_data/tablet_meta_test.hdr";
+
+    TabletMeta old_tablet_meta(1, 2, 3, 4, 5, TTabletSchema(), 6, {{7, 8}}, UniqueId(9, 10), TTabletType::TABLET_TYPE_DISK);
+    ASSERT_EQ(OLAP_SUCCESS, old_tablet_meta.save(meta_path));
+
+    {
+        // Just to make stack space dirty
+        TabletMeta new_tablet_meta;
+        new_tablet_meta._preferred_rowset_type = BETA_ROWSET;
+    }
+    TabletMeta new_tablet_meta;
+    new_tablet_meta.create_from_file(meta_path);
+
+    ASSERT_EQ(old_tablet_meta, new_tablet_meta);
+}
+
+}  // namespace doris
+
+int main(int argc, char **argv) {
+    ::testing::InitGoogleTest(&argc, argv);
+    return RUN_ALL_TESTS();
+}


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