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

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


The following commit(s) were added to refs/heads/master by this push:
     new e01986b8b9 [feature](light-schema-change) fix light-schema-change and 
add more cases (#12160)
e01986b8b9 is described below

commit e01986b8b9f1030103bc6bf4278a7ec8e3f8c893
Author: Lightman <31928846+lchangli...@users.noreply.github.com>
AuthorDate: Sat Sep 17 11:29:36 2022 +0800

    [feature](light-schema-change) fix light-schema-change and add more cases 
(#12160)
    
    Fix _delete_sign_idx and _seq_col_idx when append_column or build_schema 
when load.
    Tablet schema cache support recycle when schema sptr use count equals 1.
    Add a http interface for flink-connector to sync ddl.
    Improve tablet->tablet_schema() by max_version_schema.
---
 be/src/agent/task_worker_pool.cpp                  |   8 ++
 be/src/olap/delta_writer.cpp                       |   3 +
 be/src/olap/reader.cpp                             |   2 +-
 be/src/olap/tablet.cpp                             |  31 +++++-
 be/src/olap/tablet.h                               |  13 ++-
 be/src/olap/tablet_manager.cpp                     |   2 +-
 be/src/olap/tablet_schema.cpp                      |  13 ++-
 be/src/olap/tablet_schema_cache.h                  |  21 ++++
 be/src/vec/olap/block_reader.cpp                   |   5 +-
 be/test/olap/cumulative_compaction_policy_test.cpp |   2 +
 be/test/olap/tablet_test.cpp                       |   5 +-
 .../java/org/apache/doris/alter/RollupJobV2.java   |   2 +-
 .../apache/doris/alter/SchemaChangeHandler.java    |   3 +
 .../doris/httpv2/rest/TableSchemaAction.java       |  60 ++++++++++
 .../test_uniq_delete_sign_schema_change.out        | 123 +++++++++++++++++++++
 .../test_uniq_seq_col_schema_change.out            | 121 ++++++++++++++++++++
 .../test_uniq_delete_sign_schema_change.groovy     |  77 +++++++++++++
 .../test_uniq_seq_col_schema_change.groovy         |  78 +++++++++++++
 18 files changed, 549 insertions(+), 20 deletions(-)

diff --git a/be/src/agent/task_worker_pool.cpp 
b/be/src/agent/task_worker_pool.cpp
index b561491600..bb7e276b2d 100644
--- a/be/src/agent/task_worker_pool.cpp
+++ b/be/src/agent/task_worker_pool.cpp
@@ -856,6 +856,14 @@ void 
TaskWorkerPool::_update_tablet_meta_worker_thread_callback() {
                     if (tablet_meta_info.storage_policy.empty()) {
                         
tablet->tablet_meta()->mutable_tablet_schema()->set_is_in_memory(
                                 tablet_meta_info.is_in_memory);
+                        // The field is_in_memory should not be in the 
tablet_schema.
+                        // it should be in the tablet_meta.
+                        for (auto rowset_meta : 
tablet->tablet_meta()->all_mutable_rs_metas()) {
+                            rowset_meta->tablet_schema()->set_is_in_memory(
+                                    tablet_meta_info.is_in_memory);
+                        }
+                        
tablet->get_max_version_schema(wrlock)->set_is_in_memory(
+                                tablet_meta_info.is_in_memory);
                     } else {
                         LOG(INFO) << "set tablet cooldown resource "
                                   << tablet_meta_info.storage_policy;
diff --git a/be/src/olap/delta_writer.cpp b/be/src/olap/delta_writer.cpp
index d89d9e7023..190aac69cf 100644
--- a/be/src/olap/delta_writer.cpp
+++ b/be/src/olap/delta_writer.cpp
@@ -433,6 +433,9 @@ void DeltaWriter::_build_current_tablet_schema(int64_t 
index_id,
                                                     
ptable_schema_param.indexes(i),
                                                     ori_tablet_schema);
     }
+    if (_tablet_schema->schema_version() > ori_tablet_schema.schema_version()) 
{
+        _tablet->update_max_version_schema(_tablet_schema);
+    }
 }
 
 void DeltaWriter::_request_slave_tablet_pull_rowset(PNodeInfo node_info) {
diff --git a/be/src/olap/reader.cpp b/be/src/olap/reader.cpp
index a074ec6ffd..aaec382cb0 100644
--- a/be/src/olap/reader.cpp
+++ b/be/src/olap/reader.cpp
@@ -475,7 +475,7 @@ ColumnPredicate* TabletReader::_parse_to_predicate(
 }
 
 ColumnPredicate* TabletReader::_parse_to_predicate(const FunctionFilter& 
function_filter) {
-    int32_t index = _tablet->field_index(function_filter._col_name);
+    int32_t index = _tablet_schema->field_index(function_filter._col_name);
     if (index < 0) {
         return nullptr;
     }
diff --git a/be/src/olap/tablet.cpp b/be/src/olap/tablet.cpp
index 9a701f974b..cd1bb94c37 100644
--- a/be/src/olap/tablet.cpp
+++ b/be/src/olap/tablet.cpp
@@ -30,6 +30,7 @@
 #include <algorithm>
 #include <cstdint>
 #include <map>
+#include <memory>
 #include <mutex>
 #include <set>
 #include <shared_mutex>
@@ -99,6 +100,18 @@ Tablet::Tablet(TabletMetaSharedPtr tablet_meta, DataDir* 
data_dir,
     // construct _timestamped_versioned_tracker from rs and stale rs meta
     
_timestamped_version_tracker.construct_versioned_tracker(_tablet_meta->all_rs_metas(),
                                                              
_tablet_meta->all_stale_rs_metas());
+    // if !_tablet_meta->all_rs_metas()[0]->tablet_schema(),
+    // that mean the tablet_meta is still no upgrade to doris 1.2 versions.
+    // Before doris 1.2 version, rowset metas don't have tablet schema.
+    // And when upgrade to doris 1.2 version,
+    // all rowset metas will be set the tablet schmea from tablet meta.
+    if (_tablet_meta->all_rs_metas().empty() || 
!_tablet_meta->all_rs_metas()[0]->tablet_schema()) {
+        _max_version_schema = BaseTablet::tablet_schema();
+    } else {
+        _max_version_schema =
+                
rowset_meta_with_max_schema_version(_tablet_meta->all_rs_metas())->tablet_schema();
+    }
+    DCHECK(_max_version_schema);
 
     INT_COUNTER_METRIC_REGISTER(_metric_entity, flush_bytes);
     INT_COUNTER_METRIC_REGISTER(_metric_entity, flush_finish_count);
@@ -1865,12 +1878,20 @@ Status Tablet::remove_all_remote_rowsets() {
 
 TabletSchemaSPtr Tablet::tablet_schema() const {
     std::shared_lock wrlock(_meta_lock);
-    if (UNLIKELY(_tablet_meta->all_rs_metas().empty())) {
-        return BaseTablet::tablet_schema();
+    return _max_version_schema;
+}
+
+void Tablet::update_max_version_schema(const TabletSchemaSPtr& tablet_schema) {
+    std::lock_guard wrlock(_meta_lock);
+    // Double Check for concurrent update
+    if (!_max_version_schema ||
+        tablet_schema->schema_version() > 
_max_version_schema->schema_version()) {
+        _max_version_schema = tablet_schema;
     }
-    const RowsetMetaSharedPtr rowset_meta =
-            rowset_meta_with_max_schema_version(_tablet_meta->all_rs_metas());
-    return rowset_meta->tablet_schema();
+}
+
+TabletSchemaSPtr 
Tablet::get_max_version_schema(std::lock_guard<std::shared_mutex>&) {
+    return _max_version_schema;
 }
 
 Status Tablet::lookup_row_key(const Slice& encoded_key, const 
RowsetIdUnorderedSet* rowset_ids,
diff --git a/be/src/olap/tablet.h b/be/src/olap/tablet.h
index 128fb8d49c..7020bf0a7f 100644
--- a/be/src/olap/tablet.h
+++ b/be/src/olap/tablet.h
@@ -19,6 +19,7 @@
 
 #include <functional>
 #include <memory>
+#include <mutex>
 #include <set>
 #include <shared_mutex>
 #include <string>
@@ -112,7 +113,6 @@ public:
     double bloom_filter_fpp() const;
     size_t next_unique_id() const;
     size_t row_size() const;
-    int32_t field_index(const std::string& field_name) const;
 
     // operation in rowsets
     Status add_rowset(RowsetSharedPtr rowset);
@@ -289,6 +289,8 @@ public:
 
     TabletSchemaSPtr tablet_schema() const override;
 
+    TabletSchemaSPtr 
get_max_version_schema(std::lock_guard<std::shared_mutex>&);
+
     // Find the related rowset with specified version and return its tablet 
schema
     TabletSchemaSPtr tablet_schema(Version version) const {
         return _tablet_meta->tablet_schema(version);
@@ -348,6 +350,8 @@ public:
 
     bool check_all_rowset_segment();
 
+    void update_max_version_schema(const TabletSchemaSPtr& tablet_schema);
+
 private:
     Status _init_once_action();
     void _print_missed_versions(const std::vector<Version>& missed_versions) 
const;
@@ -457,6 +461,9 @@ private:
     // Remote rowsets not shared by other BE. We can delete them when drop 
tablet.
     std::unordered_set<RowsetSharedPtr> _self_owned_remote_rowsets; // guarded 
by _meta_lock
 
+    // Max schema_version schema from Rowset or FE
+    TabletSchemaSPtr _max_version_schema;
+
     DISALLOW_COPY_AND_ASSIGN(Tablet);
 
 public:
@@ -586,10 +593,6 @@ inline size_t Tablet::next_unique_id() const {
     return _schema->next_column_unique_id();
 }
 
-inline int32_t Tablet::field_index(const std::string& field_name) const {
-    return _schema->field_index(field_name);
-}
-
 inline size_t Tablet::row_size() const {
     return _schema->row_size();
 }
diff --git a/be/src/olap/tablet_manager.cpp b/be/src/olap/tablet_manager.cpp
index 6363418ed9..fdedd2a10d 100644
--- a/be/src/olap/tablet_manager.cpp
+++ b/be/src/olap/tablet_manager.cpp
@@ -1148,7 +1148,7 @@ Status TabletManager::_create_tablet_meta_unlocked(const 
TCreateTabletReq& reque
             //    unique_id in old_tablet to be the column's ordinal number in 
new_tablet
             // 2. if column exists only in new_tablet, assign next_unique_id 
of old_tablet
             //    to the new column
-            int32_t old_col_idx = base_tablet->field_index(column.column_name);
+            int32_t old_col_idx = 
base_tablet->tablet_schema()->field_index(column.column_name);
             if (old_col_idx != -1) {
                 uint32_t old_unique_id =
                         
base_tablet->tablet_schema()->column(old_col_idx).unique_id();
diff --git a/be/src/olap/tablet_schema.cpp b/be/src/olap/tablet_schema.cpp
index 70c06adab9..691e72e0bf 100644
--- a/be/src/olap/tablet_schema.cpp
+++ b/be/src/olap/tablet_schema.cpp
@@ -20,6 +20,7 @@
 #include <gen_cpp/olap_file.pb.h>
 
 #include "gen_cpp/descriptors.pb.h"
+#include "olap/utils.h"
 #include "tablet_meta.h"
 #include "vec/aggregate_functions/aggregate_function_reader.h"
 #include "vec/aggregate_functions/aggregate_function_simple_factory.h"
@@ -465,6 +466,11 @@ void TabletSchema::append_column(TabletColumn column, bool 
is_dropped_column) {
     if (column.is_nullable()) {
         _num_null_columns++;
     }
+    if (UNLIKELY(column.name() == DELETE_SIGN)) {
+        _delete_sign_idx = _num_columns;
+    } else if (UNLIKELY(column.name() == SEQUENCE_COL)) {
+        _sequence_col_idx = _num_columns;
+    }
     // The dropped column may have same name with exsiting column, so that
     // not add to name to index map, only for uid to index map
     if (!is_dropped_column) {
@@ -552,8 +558,6 @@ void TabletSchema::build_current_tablet_schema(int64_t 
index_id, int32_t version
     _next_column_unique_id = ori_tablet_schema.next_column_unique_id();
     _is_in_memory = ori_tablet_schema.is_in_memory();
     _disable_auto_compaction = ori_tablet_schema.disable_auto_compaction();
-    _delete_sign_idx = ori_tablet_schema.delete_sign_idx();
-    _sequence_col_idx = ori_tablet_schema.sequence_col_idx();
     _sort_type = ori_tablet_schema.sort_type();
     _sort_col_num = ori_tablet_schema.sort_col_num();
 
@@ -579,6 +583,11 @@ void TabletSchema::build_current_tablet_schema(int64_t 
index_id, int32_t version
         if (column.is_bf_column()) {
             has_bf_columns = true;
         }
+        if (UNLIKELY(column.name() == DELETE_SIGN)) {
+            _delete_sign_idx = _num_columns;
+        } else if (UNLIKELY(column.name() == SEQUENCE_COL)) {
+            _sequence_col_idx = _num_columns;
+        }
         _field_name_to_index[column.name()] = _num_columns;
         _field_id_to_index[column.unique_id()] = _num_columns;
         _cols.emplace_back(std::move(column));
diff --git a/be/src/olap/tablet_schema_cache.h 
b/be/src/olap/tablet_schema_cache.h
index 5e530bf08c..4dbdd93c6a 100644
--- a/be/src/olap/tablet_schema_cache.h
+++ b/be/src/olap/tablet_schema_cache.h
@@ -33,6 +33,8 @@ public:
         DCHECK(_s_instance == nullptr);
         static TabletSchemaCache instance;
         _s_instance = &instance;
+        std::thread t(&TabletSchemaCache::_recycle, _s_instance);
+        t.detach();
     }
 
     static TabletSchemaCache* instance() { return _s_instance; }
@@ -52,6 +54,25 @@ public:
         return iter->second;
     }
 
+private:
+    /**
+     * @brief recycle when TabletSchemaSPtr use_count equals 1.
+     */
+    void _recycle() {
+        int64_t tablet_schema_cache_recycle_interval = 86400; // s, one day
+        for (;;) {
+            
std::this_thread::sleep_for(std::chrono::seconds(tablet_schema_cache_recycle_interval));
+            std::lock_guard guard(_mtx);
+            for (auto iter = _cache.begin(), last = _cache.end(); iter != 
last;) {
+                if (iter->second.unique()) {
+                    iter = _cache.erase(iter);
+                } else {
+                    ++iter;
+                }
+            }
+        }
+    }
+
 private:
     static inline TabletSchemaCache* _s_instance = nullptr;
     std::mutex _mtx;
diff --git a/be/src/vec/olap/block_reader.cpp b/be/src/vec/olap/block_reader.cpp
index 41b54cea4d..570315b431 100644
--- a/be/src/vec/olap/block_reader.cpp
+++ b/be/src/vec/olap/block_reader.cpp
@@ -283,8 +283,7 @@ Status BlockReader::_unique_key_next_block(Block* block, 
MemPool* mem_pool, Obje
 
     // do filter detete row in base compaction, only base compaction need to 
do the job
     if (_filter_delete) {
-        int delete_sign_idx =
-                (_sequence_col_idx == -1) ? target_columns.size() - 1 : 
target_columns.size() - 2;
+        int delete_sign_idx = 
_reader_context.tablet_schema->field_index(DELETE_SIGN);
         DCHECK(delete_sign_idx > 0);
         MutableColumnPtr delete_filter_column = 
(*std::move(_delete_filter_column)).mutate();
         
reinterpret_cast<ColumnUInt8*>(delete_filter_column.get())->resize(target_block_row);
@@ -414,7 +413,7 @@ void BlockReader::_update_agg_value(MutableColumns& 
columns, int begin, int end,
 }
 
 ColumnPredicate* BlockReader::_parse_to_predicate(const FunctionFilter& 
function_filter) {
-    int32_t index = _tablet->field_index(function_filter._col_name);
+    int32_t index = _tablet_schema->field_index(function_filter._col_name);
     if (index < 0) {
         return nullptr;
     }
diff --git a/be/test/olap/cumulative_compaction_policy_test.cpp 
b/be/test/olap/cumulative_compaction_policy_test.cpp
index a7117fba43..1c25041dd8 100644
--- a/be/test/olap/cumulative_compaction_policy_test.cpp
+++ b/be/test/olap/cumulative_compaction_policy_test.cpp
@@ -64,6 +64,7 @@ public:
         pb1->set_start_version(start);
         pb1->set_end_version(end);
         pb1->set_creation_time(10000);
+        pb1->set_tablet_schema(_tablet_meta->tablet_schema());
     }
 
     void init_all_rs_meta(std::vector<RowsetMetaSharedPtr>* rs_metas) {
@@ -304,6 +305,7 @@ public:
         pb1->set_end_version(end);
         pb1->set_total_disk_size(41);
         pb1->set_creation_time(10000);
+        pb1->set_tablet_schema(_tablet_meta->tablet_schema());
     }
 
     void init_rs_meta_small_base(std::vector<RowsetMetaSharedPtr>* rs_metas) {
diff --git a/be/test/olap/tablet_test.cpp b/be/test/olap/tablet_test.cpp
index 197a53666c..6e9990dd3b 100644
--- a/be/test/olap/tablet_test.cpp
+++ b/be/test/olap/tablet_test.cpp
@@ -118,6 +118,7 @@ public:
         pb1->set_start_version(start);
         pb1->set_end_version(end);
         pb1->set_creation_time(10000);
+        pb1->set_tablet_schema(_tablet_meta->tablet_schema());
     }
 
     void init_rs_meta(RowsetMetaSharedPtr& pb1, int64_t start, int64_t end, 
int64_t earliest_ts,
@@ -129,6 +130,7 @@ public:
         pb1->set_end_version(end);
         pb1->set_creation_time(10000);
         pb1->set_num_segments(2);
+        pb1->set_tablet_schema(_tablet_meta->tablet_schema());
     }
 
     void init_rs_meta(RowsetMetaSharedPtr& pb1, int64_t start, int64_t end,
@@ -139,6 +141,7 @@ public:
         pb1->set_creation_time(10000);
         pb1->set_segments_key_bounds(keybounds);
         pb1->set_num_segments(keybounds.size());
+        pb1->set_tablet_schema(_tablet_meta->tablet_schema());
     }
 
     void init_all_rs_meta(std::vector<RowsetMetaSharedPtr>* rs_metas) {
@@ -381,7 +384,6 @@ TEST_F(TestTablet, rowset_tree_update) {
 
     RowsetMetaSharedPtr rsm1(new RowsetMeta());
     init_rs_meta(rsm1, 6, 7, convert_key_bounds({{"100", "200"}, {"300", 
"400"}}));
-    rsm1->set_tablet_schema(tablet->tablet_schema());
     RowsetId id1;
     id1.init(10010);
     RowsetSharedPtr rs_ptr1;
@@ -391,7 +393,6 @@ TEST_F(TestTablet, rowset_tree_update) {
 
     RowsetMetaSharedPtr rsm2(new RowsetMeta());
     init_rs_meta(rsm2, 8, 8, convert_key_bounds({{"500", "999"}}));
-    rsm2->set_tablet_schema(tablet->tablet_schema());
     RowsetId id2;
     id2.init(10086);
     rsm2->set_rowset_id(id2);
diff --git a/fe/fe-core/src/main/java/org/apache/doris/alter/RollupJobV2.java 
b/fe/fe-core/src/main/java/org/apache/doris/alter/RollupJobV2.java
index 8d75cd774d..3ca0ad2e0e 100644
--- a/fe/fe-core/src/main/java/org/apache/doris/alter/RollupJobV2.java
+++ b/fe/fe-core/src/main/java/org/apache/doris/alter/RollupJobV2.java
@@ -402,7 +402,7 @@ public class RollupJobV2 extends AlterJobV2 implements 
GsonPostProcessable {
                         AlterReplicaTask rollupTask = new 
AlterReplicaTask(rollupReplica.getBackendId(), dbId, tableId,
                                 partitionId, rollupIndexId, baseIndexId, 
rollupTabletId, baseTabletId,
                                 rollupReplica.getId(), rollupSchemaHash, 
baseSchemaHash, visibleVersion, jobId,
-                                JobType.ROLLUP, defineExprs, descTable, 
tbl.getSchemaByIndexId(baseIndexId));
+                                JobType.ROLLUP, defineExprs, descTable, 
tbl.getSchemaByIndexId(baseIndexId, true));
                         rollupBatchTask.addTask(rollupTask);
                     }
                 }
diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/alter/SchemaChangeHandler.java 
b/fe/fe-core/src/main/java/org/apache/doris/alter/SchemaChangeHandler.java
index c1955acb95..85e12954c8 100644
--- a/fe/fe-core/src/main/java/org/apache/doris/alter/SchemaChangeHandler.java
+++ b/fe/fe-core/src/main/java/org/apache/doris/alter/SchemaChangeHandler.java
@@ -386,6 +386,9 @@ public class SchemaChangeHandler extends AlterHandler {
                 while (iter.hasNext()) {
                     Column column = iter.next();
                     if (column.getName().equalsIgnoreCase(dropColName)) {
+                        if (column.isKey()) {
+                            lightSchemaChange = false;
+                        }
                         iter.remove();
                         break;
                     }
diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/httpv2/rest/TableSchemaAction.java 
b/fe/fe-core/src/main/java/org/apache/doris/httpv2/rest/TableSchemaAction.java
index d05fd3d728..ae1b395617 100644
--- 
a/fe/fe-core/src/main/java/org/apache/doris/httpv2/rest/TableSchemaAction.java
+++ 
b/fe/fe-core/src/main/java/org/apache/doris/httpv2/rest/TableSchemaAction.java
@@ -31,13 +31,17 @@ import org.apache.doris.httpv2.entity.ResponseEntityBuilder;
 import org.apache.doris.mysql.privilege.PrivPredicate;
 import org.apache.doris.qe.ConnectContext;
 
+import com.google.gson.Gson;
+import com.google.gson.reflect.TypeToken;
 import org.springframework.web.bind.annotation.PathVariable;
+import org.springframework.web.bind.annotation.RequestBody;
 import org.springframework.web.bind.annotation.RequestMapping;
 import org.springframework.web.bind.annotation.RequestMethod;
 import org.springframework.web.bind.annotation.RestController;
 
 import java.util.ArrayList;
 import java.util.HashMap;
+import java.util.Iterator;
 import java.util.List;
 import java.util.Map;
 import java.util.Optional;
@@ -110,4 +114,60 @@ public class TableSchemaAction extends RestBaseController {
 
         return ResponseEntityBuilder.ok(resultMap);
     }
+
+    private static class DDLRequestBody {
+        public Boolean isDropColumn;
+        public String columnName;
+    }
+
+    /**
+     * Request body:
+     * {
+     * "isDropColumn": 1,   // optional
+     * "columnName" : "value1"   // optional
+     * }
+     */
+    @RequestMapping(path = "/api/enable_light_schema_change/{" + DB_KEY
+                    + "}/{" + TABLE_KEY + "}", method = { RequestMethod.GET })
+    public Object columnChangeCanSync(
+            @PathVariable(value = DB_KEY) String dbName,
+            @PathVariable(value = TABLE_KEY) String tableName,
+            HttpServletRequest request, HttpServletResponse response, 
@RequestBody String body) {
+        executeCheckPassword(request, response);
+        String fullDbName = getFullDbName(dbName);
+        OlapTable table;
+        try {
+            Database db = 
Env.getCurrentInternalCatalog().getDbOrMetaException(fullDbName);
+            table = (OlapTable) db.getTableOrMetaException(tableName, 
Table.TableType.OLAP);
+        } catch (MetaNotFoundException e) {
+            return ResponseEntityBuilder.okWithCommonError(e.getMessage());
+        }
+        if (!table.getEnableLightSchemaChange()) {
+            return ResponseEntityBuilder.okWithCommonError("table " + 
tableName + " disable light schema change");
+        }
+        java.lang.reflect.Type type = new TypeToken<DDLRequestBody>() 
{}.getType();
+        DDLRequestBody ddlRequestBody = new Gson().fromJson(body, type);
+        if (ddlRequestBody.isDropColumn) {
+            boolean enableLightSchemaChange = true;
+            // column should be dropped from both base and rollup indexes.
+            for (Map.Entry<Long, List<Column>> entry : 
table.getIndexIdToSchema().entrySet()) {
+                List<Column> baseSchema = entry.getValue();
+                Iterator<Column> baseIter = baseSchema.iterator();
+                while (baseIter.hasNext()) {
+                    Column column = baseIter.next();
+                    if 
(column.getName().equalsIgnoreCase(ddlRequestBody.columnName)) {
+                        if (column.isKey()) {
+                            enableLightSchemaChange = false;
+                        }
+                        break;
+                    }
+                }
+            }
+            if (!enableLightSchemaChange) {
+                return ResponseEntityBuilder.okWithCommonError("Column " + 
ddlRequestBody.columnName
+                                + " is primary key in materializedIndex that 
can't do the light schema change");
+            }
+        }
+        return ResponseEntityBuilder.ok();
+    }
 }
diff --git 
a/regression-test/data/schema_change_p0/test_uniq_delete_sign_schema_change.out 
b/regression-test/data/schema_change_p0/test_uniq_delete_sign_schema_change.out
new file mode 100644
index 0000000000..08b85106f7
--- /dev/null
+++ 
b/regression-test/data/schema_change_p0/test_uniq_delete_sign_schema_change.out
@@ -0,0 +1,123 @@
+-- This file is automatically generated. You should know what you did if you 
want to edit this
+-- !sql --
+1      1       1       2
+2      2       2       2
+3      3       3       3
+4      4       4       4
+
+-- !sql --
+1      1       1       2
+2      2       2       2
+3      3       3       3
+4      4       4       4
+
+-- !sql --
+1      1       1       1       1
+2      2       2       2       0
+3      3       3       3       0
+4      4       4       4       0
+
+-- !sql --
+1      1       1       1       1
+2      2       2       2       0
+3      3       3       3       0
+4      4       4       4       0
+
+-- !sql --
+k1     INT     Yes     true    \N      
+value1 INT     Yes     false   \N      REPLACE
+value2 INT     Yes     false   \N      REPLACE
+value3 INT     Yes     false   \N      REPLACE
+value4 INT     Yes     false   \N      REPLACE
+__DORIS_DELETE_SIGN__  TINYINT No      false   0       REPLACE
+
+-- !sql --
+1      1       1       1       \N      1
+2      2       2       2       \N      0
+3      3       3       3       \N      0
+4      4       4       4       \N      0
+
+-- !sql --
+1      1       1       1       \N      1
+2      2       2       2       \N      0
+3      3       3       3       \N      0
+4      4       4       4       \N      0
+
+-- !sql --
+1      1       1       1       \N      1
+2      2       2       2       \N      0
+3      3       3       3       \N      0
+4      4       4       4       \N      0
+5      5       5       5       5       0
+6      6       6       6       6       0
+
+-- !sql --
+1      1       1       1       \N      1
+2      2       2       2       \N      0
+3      3       3       3       \N      0
+4      4       4       4       \N      0
+5      5       5       5       5       0
+6      6       6       6       6       0
+
+-- !sql --
+1      1       1       1       \N      1
+2      2       2       2       \N      0
+3      1       1       1       1       1
+4      4       4       4       \N      0
+5      1       1       1       1       1
+6      6       6       6       6       0
+
+-- !sql --
+1      1       1       1       \N      1
+2      2       2       2       \N      0
+3      1       1       1       1       1
+4      4       4       4       \N      0
+5      1       1       1       1       1
+6      6       6       6       6       0
+
+-- !sql --
+k1     INT     Yes     true    \N      
+value2 INT     Yes     false   \N      REPLACE
+value4 INT     Yes     false   \N      REPLACE
+__DORIS_DELETE_SIGN__  TINYINT No      false   0       REPLACE
+
+-- !sql --
+1      1       \N      1
+2      2       \N      0
+3      1       1       1
+4      4       \N      0
+5      1       1       1
+6      6       6       0
+7      7       7       0
+
+-- !sql --
+1      1       \N      1
+2      2       \N      0
+3      1       1       1
+4      4       \N      0
+5      1       1       1
+6      6       6       0
+7      7       7       0
+
+-- !sql --
+1      1       \N      1
+2      1       1       1
+3      1       1       1
+4      1       1       1
+5      1       1       1
+6      1       1       1
+7      1       1       1
+
+-- !sql --
+1      1       \N      1
+2      1       1       1
+3      1       1       1
+4      1       1       1
+5      1       1       1
+6      1       1       1
+7      1       1       1
+
+-- !sql --
+
+-- !sql --
+
diff --git 
a/regression-test/data/schema_change_p0/test_uniq_seq_col_schema_change.out 
b/regression-test/data/schema_change_p0/test_uniq_seq_col_schema_change.out
new file mode 100644
index 0000000000..563106c144
--- /dev/null
+++ b/regression-test/data/schema_change_p0/test_uniq_seq_col_schema_change.out
@@ -0,0 +1,121 @@
+-- This file is automatically generated. You should know what you did if you 
want to edit this
+-- !sql --
+1      1       1       2
+2      2       2       2
+3      3       3       3
+4      4       4       4
+
+-- !sql --
+1      1       1       2
+2      2       2       2
+3      3       3       3
+4      4       4       4
+
+-- !sql --
+1      1       1       2       \N
+2      2       2       2       \N
+3      3       3       3       \N
+4      4       4       4       \N
+
+-- !sql --
+1      1       1       2       \N
+2      2       2       2       \N
+3      3       3       3       \N
+4      4       4       4       \N
+
+-- !sql --
+1      1       1       2       \N
+2      2       2       2       \N
+3      3       3       3       \N
+4      4       4       4       \N
+5      6       6       6       6
+
+-- !sql --
+1      1       1       2       \N
+2      2       2       2       \N
+3      3       3       3       \N
+4      4       4       4       \N
+5      6       6       6       6
+
+-- !sql --
+1      1       1       2       \N
+2      2       2       2       \N
+3      3       3       3       \N
+4      4       4       4       \N
+5      6       6       7       6
+
+-- !sql --
+1      1       1       2       \N
+2      2       2       2       \N
+3      3       3       3       \N
+4      4       4       4       \N
+5      6       6       7       6
+
+-- !sql --
+1      1       \N
+2      2       \N
+3      3       \N
+4      4       \N
+5      6       6
+
+-- !sql --
+1      1       \N
+2      2       \N
+3      3       \N
+4      4       \N
+5      6       6
+
+-- !sql --
+1      1       \N
+2      2       \N
+3      3       \N
+4      4       \N
+5      6       6
+6      6       6
+7      2       1
+
+-- !sql --
+1      1       \N
+2      2       \N
+3      3       \N
+4      4       \N
+5      6       6
+6      6       6
+7      2       1
+
+-- !sql --
+1      1       \N
+2      2       \N
+3      3       \N
+4      4       \N
+5      6       6
+6      6       6
+7      2       1
+
+-- !sql --
+1      1       \N
+2      2       \N
+3      3       \N
+4      4       \N
+5      6       6
+6      6       6
+7      2       1
+
+-- !sql --
+1      1       \N
+2      3       2
+3      3       \N
+4      4       \N
+5      6       6
+6      6       6
+7      2       1
+
+-- !sql --
+1      1       \N
+2      3       2
+3      3       \N
+4      4       \N
+5      6       6
+6      6       6
+7      2       1
+
diff --git 
a/regression-test/suites/schema_change_p0/test_uniq_delete_sign_schema_change.groovy
 
b/regression-test/suites/schema_change_p0/test_uniq_delete_sign_schema_change.groovy
new file mode 100644
index 0000000000..ffd2b37c26
--- /dev/null
+++ 
b/regression-test/suites/schema_change_p0/test_uniq_delete_sign_schema_change.groovy
@@ -0,0 +1,77 @@
+// 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("test_uniq_delete_sign_schema_change", "schema_change") {
+    def tbName1 = "test_uniq_delete_sign_schema_change"
+
+    sql "DROP TABLE IF EXISTS ${tbName1}"
+    sql """
+            CREATE TABLE IF NOT EXISTS ${tbName1} (
+                k1 INT,
+                value1 INT,
+                value2 INT,
+                value3 INT
+            )
+            UNIQUE KEY (k1)
+            DISTRIBUTED BY HASH(k1) BUCKETS 1 properties("replication_num" = 
"1", "light_schema_change" = "true");
+        """
+    sql "insert into ${tbName1} values(1,1,1,1);"
+    sql "insert into ${tbName1} values(1,1,1,2);"
+    sql "insert into ${tbName1} values(2,2,2,2);"
+    sql "insert into ${tbName1} values(3,3,3,3);"
+    sql "insert into ${tbName1} values(4,4,4,4);"
+    qt_sql "select /*+ SET_VAR(enable_vectorized_engine=true) */ * from 
${tbName1} order by k1;"
+    qt_sql "select /*+ SET_VAR(enable_vectorized_engine=false) */ * from 
${tbName1} order by k1;"
+
+    sql "set show_hidden_columns = true"
+    sql "insert into ${tbName1} (k1, value1, value2, value3, 
__DORIS_DELETE_SIGN__) values(1,1,1,1,1);"
+    qt_sql "select /*+ SET_VAR(enable_vectorized_engine=true) */ * from 
${tbName1} order by k1;"
+    qt_sql "select /*+ SET_VAR(enable_vectorized_engine=false) */ * from 
${tbName1} order by k1;"
+
+    sql "ALTER TABLE ${tbName1} ADD COLUMN value4 INT;"
+    qt_sql "desc ${tbName1};"
+    qt_sql "select /*+ SET_VAR(enable_vectorized_engine=true) */ * from 
${tbName1} order by k1;"
+    qt_sql "select /*+ SET_VAR(enable_vectorized_engine=false) */ * from 
${tbName1} order by k1;"
+
+    sql "insert into ${tbName1} values(5,5,5,5,5);"
+    sql "insert into ${tbName1} values(6,6,6,6,6);"
+    qt_sql "select /*+ SET_VAR(enable_vectorized_engine=true) */ * from 
${tbName1} order by k1;"
+    qt_sql "select /*+ SET_VAR(enable_vectorized_engine=false) */ * from 
${tbName1} order by k1;"
+
+    sql "insert into ${tbName1} (k1, value1, value2, value3, value4, 
__DORIS_DELETE_SIGN__) values(5,1,1,1,1,1);"
+    sql "insert into ${tbName1} (k1, value1, value2, value3, value4, 
__DORIS_DELETE_SIGN__) values(3,1,1,1,1,1);"
+    qt_sql "select /*+ SET_VAR(enable_vectorized_engine=true) */ * from 
${tbName1} order by k1;"
+    qt_sql "select /*+ SET_VAR(enable_vectorized_engine=false) */ * from 
${tbName1} order by k1;"
+
+    sql "ALTER TABLE ${tbName1} DROP COLUMN value1;"
+    sql "ALTER TABLE ${tbName1} DROP COLUMN value3;"
+    qt_sql "desc ${tbName1};"
+    sql "insert into ${tbName1} values(7,7,7);"
+    qt_sql "select /*+ SET_VAR(enable_vectorized_engine=true) */ * from 
${tbName1} order by k1;"
+    qt_sql "select /*+ SET_VAR(enable_vectorized_engine=false) */ * from 
${tbName1} order by k1;"
+
+    sql "insert into ${tbName1} (k1, value2, value4, __DORIS_DELETE_SIGN__) 
values(2,1,1,1);"
+    sql "insert into ${tbName1} (k1, value2, value4, __DORIS_DELETE_SIGN__) 
values(4,1,1,1);"
+    sql "insert into ${tbName1} (k1, value2, value4, __DORIS_DELETE_SIGN__) 
values(6,1,1,1);"
+    sql "insert into ${tbName1} (k1, value2, value4, __DORIS_DELETE_SIGN__) 
values(7,1,1,1);"
+    qt_sql "select /*+ SET_VAR(enable_vectorized_engine=true) */ * from 
${tbName1} order by k1;"
+    qt_sql "select /*+ SET_VAR(enable_vectorized_engine=false) */ * from 
${tbName1} order by k1;"
+
+    sql "set show_hidden_columns = false"
+    qt_sql "select /*+ SET_VAR(enable_vectorized_engine=true) */ * from 
${tbName1} order by k1;"
+    qt_sql "select /*+ SET_VAR(enable_vectorized_engine=false) */ * from 
${tbName1} order by k1;"
+}
diff --git 
a/regression-test/suites/schema_change_p0/test_uniq_seq_col_schema_change.groovy
 
b/regression-test/suites/schema_change_p0/test_uniq_seq_col_schema_change.groovy
new file mode 100644
index 0000000000..b48e4dc742
--- /dev/null
+++ 
b/regression-test/suites/schema_change_p0/test_uniq_seq_col_schema_change.groovy
@@ -0,0 +1,78 @@
+// 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("test_uniq_seq_col_schema_change", "schema_change") {
+    def tbName1 = "test_uniq_seq_col_schema_change"
+    def columnWithHidden_1 = 
"(k1,value1,value2,value3,__DORIS_DELETE_SIGN__,__DORIS_SEQUENCE_COL__)"
+    def columnWithHidden_2 = 
"(k1,value1,value2,value3,value4,__DORIS_DELETE_SIGN__,__DORIS_SEQUENCE_COL__)"
+    def columnWithHidden_3 = 
"(k1,value2,value4,__DORIS_DELETE_SIGN__,__DORIS_SEQUENCE_COL__)"
+    sql "DROP TABLE IF EXISTS ${tbName1};"
+    sql """
+            CREATE TABLE IF NOT EXISTS ${tbName1} (
+                k1 INT,
+                value1 INT,
+                value2 INT,
+                value3 INT
+            )
+            UNIQUE KEY (k1)
+            DISTRIBUTED BY HASH(k1) BUCKETS 1 
+            properties("replication_num" = "1",
+                       "light_schema_change" = "true",
+                       "function_column.sequence_type" = 'INT');
+        """
+        // sequence col equal value3
+        sql "insert into ${tbName1} ${columnWithHidden_1} values(1,1,1,2,0,2);"
+        sql "insert into ${tbName1} ${columnWithHidden_1} values(1,1,1,1,0,1);"
+        sql "insert into ${tbName1} ${columnWithHidden_1} values(2,2,2,2,0,2);"
+        sql "insert into ${tbName1} ${columnWithHidden_1} values(3,3,3,3,0,3);"
+        sql "insert into ${tbName1} ${columnWithHidden_1} values(4,4,4,4,0,4);"
+        qt_sql "select /*+ SET_VAR(enable_vectorized_engine=true) */ * from 
${tbName1} order by k1;"
+        qt_sql "select /*+ SET_VAR(enable_vectorized_engine=false) */ * from 
${tbName1} order by k1;"
+
+        sql "ALTER TABLE ${tbName1} ADD COLUMN value4 INT;"
+        qt_sql "select /*+ SET_VAR(enable_vectorized_engine=true) */ * from 
${tbName1} order by k1;"
+        qt_sql "select /*+ SET_VAR(enable_vectorized_engine=false) */ * from 
${tbName1} order by k1;"
+
+        sql "insert into ${tbName1} 
${columnWithHidden_2}values(5,5,5,5,5,0,5);"
+        sql "insert into ${tbName1} 
${columnWithHidden_2}values(5,6,6,6,6,0,6);"
+        sql "insert into ${tbName1} values(5,6,6,7,6);"
+        qt_sql "select /*+ SET_VAR(enable_vectorized_engine=true) */ * from 
${tbName1} order by k1;"
+        qt_sql "select /*+ SET_VAR(enable_vectorized_engine=false) */ * from 
${tbName1} order by k1;"
+        sql "insert into ${tbName1} 
${columnWithHidden_2}values(5,6,6,7,6,0,7);"
+        qt_sql "select /*+ SET_VAR(enable_vectorized_engine=true) */ * from 
${tbName1} order by k1;"
+        qt_sql "select /*+ SET_VAR(enable_vectorized_engine=false) */ * from 
${tbName1} order by k1;"
+
+        sql "ALTER TABLE ${tbName1} DROP COLUMN value1;"
+        sql "ALTER TABLE ${tbName1} DROP COLUMN value3;"
+        qt_sql "select /*+ SET_VAR(enable_vectorized_engine=true) */ * from 
${tbName1} order by k1;"
+        qt_sql "select /*+ SET_VAR(enable_vectorized_engine=false) */ * from 
${tbName1} order by k1;"
+
+        // sequence col equal value2
+        sql "insert into ${tbName1} ${columnWithHidden_3} values(6,6,6,0,6);"
+        sql "insert into ${tbName1} ${columnWithHidden_3} values(6,5,6,0,5);"
+        sql "insert into ${tbName1} ${columnWithHidden_3} values(7,1,1,0,1);"
+        sql "insert into ${tbName1} ${columnWithHidden_3} values(7,2,1,0,2);"
+        qt_sql "select /*+ SET_VAR(enable_vectorized_engine=true) */ * from 
${tbName1} order by k1;"
+        qt_sql "select /*+ SET_VAR(enable_vectorized_engine=false) */ * from 
${tbName1} order by k1;"
+
+        sql "insert into ${tbName1} ${columnWithHidden_3} values(2,1,2,0,1);"
+        qt_sql "select /*+ SET_VAR(enable_vectorized_engine=true) */ * from 
${tbName1} order by k1;"
+        qt_sql "select /*+ SET_VAR(enable_vectorized_engine=false) */ * from 
${tbName1} order by k1;"
+        sql "insert into ${tbName1} ${columnWithHidden_3} values(2,3,2,0,3);"
+        qt_sql "select /*+ SET_VAR(enable_vectorized_engine=true) */ * from 
${tbName1} order by k1;"
+        qt_sql "select /*+ SET_VAR(enable_vectorized_engine=false) */ * from 
${tbName1} order by k1;"
+}


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


Reply via email to