This is an automated email from the ASF dual-hosted git repository. dataroaring pushed a commit to branch branch-3.0 in repository https://gitbox.apache.org/repos/asf/doris.git
The following commit(s) were added to refs/heads/branch-3.0 by this push: new 6a7a61da1f9 branch-3.0: [feature](restore) support force_replace restore#47314 (#48050) 6a7a61da1f9 is described below commit 6a7a61da1f938cc9f41dffcdc660f62ed81a9499 Author: Uniqueyou <wangyix...@selectdb.com> AuthorDate: Wed Feb 19 19:26:00 2025 +0800 branch-3.0: [feature](restore) support force_replace restore#47314 (#48050) pick: https://github.com/apache/doris/pull/47314 --- .../org/apache/doris/analysis/RestoreStmt.java | 9 ++ .../org/apache/doris/backup/BackupHandler.java | 12 ++- .../java/org/apache/doris/backup/RestoreJob.java | 99 +++++++++++++------- .../apache/doris/service/FrontendServiceImpl.java | 3 + .../org/apache/doris/backup/RestoreJobTest.java | 3 +- gensrc/thrift/FrontendService.thrift | 1 + ...backup_restore_force_replace_diff_column.groovy | 96 ++++++++++++++++++++ ...kup_restore_force_replace_diff_part_type.groovy | 97 ++++++++++++++++++++ ...ckup_restore_force_replace_diff_part_val.groovy | 101 +++++++++++++++++++++ 9 files changed, 384 insertions(+), 37 deletions(-) diff --git a/fe/fe-core/src/main/java/org/apache/doris/analysis/RestoreStmt.java b/fe/fe-core/src/main/java/org/apache/doris/analysis/RestoreStmt.java index bc38cfe09e5..5f141837565 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/analysis/RestoreStmt.java +++ b/fe/fe-core/src/main/java/org/apache/doris/analysis/RestoreStmt.java @@ -44,6 +44,7 @@ public class RestoreStmt extends AbstractBackupStmt implements NotFallbackInPars public static final String PROP_CLEAN_TABLES = "clean_tables"; public static final String PROP_CLEAN_PARTITIONS = "clean_partitions"; public static final String PROP_ATOMIC_RESTORE = "atomic_restore"; + public static final String PROP_FORCE_REPLACE = "force_replace"; private boolean allowLoad = false; private ReplicaAllocation replicaAlloc = ReplicaAllocation.DEFAULT_ALLOCATION; @@ -56,6 +57,7 @@ public class RestoreStmt extends AbstractBackupStmt implements NotFallbackInPars private boolean isCleanTables = false; private boolean isCleanPartitions = false; private boolean isAtomicRestore = false; + private boolean isForceReplace = false; private byte[] meta = null; private byte[] jobInfo = null; @@ -127,6 +129,10 @@ public class RestoreStmt extends AbstractBackupStmt implements NotFallbackInPars return isAtomicRestore; } + public boolean isForceReplace() { + return isForceReplace; + } + @Override public void analyze(Analyzer analyzer) throws UserException { if (repoName.equals(Repository.KEEP_ON_LOCAL_REPO_NAME)) { @@ -212,6 +218,9 @@ public class RestoreStmt extends AbstractBackupStmt implements NotFallbackInPars // is atomic restore isAtomicRestore = eatBooleanProperty(copiedProperties, PROP_ATOMIC_RESTORE, isAtomicRestore); + // is force replace + isForceReplace = eatBooleanProperty(copiedProperties, PROP_FORCE_REPLACE, isForceReplace); + if (!copiedProperties.isEmpty()) { ErrorReport.reportAnalysisException(ErrorCode.ERR_COMMON_ERROR, "Unknown restore job properties: " + copiedProperties.keySet()); diff --git a/fe/fe-core/src/main/java/org/apache/doris/backup/BackupHandler.java b/fe/fe-core/src/main/java/org/apache/doris/backup/BackupHandler.java index a6217501987..9a709fa0323 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/backup/BackupHandler.java +++ b/fe/fe-core/src/main/java/org/apache/doris/backup/BackupHandler.java @@ -555,14 +555,16 @@ public class BackupHandler extends MasterDaemon implements Writable { db.getId(), db.getFullName(), jobInfo, stmt.allowLoad(), stmt.getReplicaAlloc(), stmt.getTimeoutMs(), metaVersion, stmt.reserveReplica(), stmt.reserveDynamicPartitionEnable(), stmt.isBeingSynced(), - stmt.isCleanTables(), stmt.isCleanPartitions(), stmt.isAtomicRestore(), + stmt.isCleanTables(), stmt.isCleanPartitions(), stmt.isAtomicRestore(), stmt.isForceReplace(), env, Repository.KEEP_ON_LOCAL_REPO_ID, backupMeta); } else { restoreJob = new RestoreJob(stmt.getLabel(), stmt.getBackupTimestamp(), - db.getId(), db.getFullName(), jobInfo, stmt.allowLoad(), stmt.getReplicaAlloc(), - stmt.getTimeoutMs(), stmt.getMetaVersion(), stmt.reserveReplica(), stmt.reserveDynamicPartitionEnable(), - stmt.isBeingSynced(), stmt.isCleanTables(), stmt.isCleanPartitions(), stmt.isAtomicRestore(), - env, repository.getId()); + db.getId(), db.getFullName(), jobInfo, stmt.allowLoad(), stmt.getReplicaAlloc(), + stmt.getTimeoutMs(), stmt.getMetaVersion(), stmt.reserveReplica(), + stmt.reserveDynamicPartitionEnable(), + stmt.isBeingSynced(), stmt.isCleanTables(), stmt.isCleanPartitions(), stmt.isAtomicRestore(), + stmt.isForceReplace(), + env, repository.getId()); } env.getEditLog().logRestoreJob(restoreJob); diff --git a/fe/fe-core/src/main/java/org/apache/doris/backup/RestoreJob.java b/fe/fe-core/src/main/java/org/apache/doris/backup/RestoreJob.java index c3f242143d0..55c7eee0d44 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/backup/RestoreJob.java +++ b/fe/fe-core/src/main/java/org/apache/doris/backup/RestoreJob.java @@ -124,6 +124,7 @@ public class RestoreJob extends AbstractJob implements GsonPostProcessable { private static final String PROP_CLEAN_TABLES = RestoreStmt.PROP_CLEAN_TABLES; private static final String PROP_CLEAN_PARTITIONS = RestoreStmt.PROP_CLEAN_PARTITIONS; private static final String PROP_ATOMIC_RESTORE = RestoreStmt.PROP_ATOMIC_RESTORE; + private static final String PROP_FORCE_REPLACE = RestoreStmt.PROP_FORCE_REPLACE; private static final String ATOMIC_RESTORE_TABLE_PREFIX = "__doris_atomic_restore_prefix__"; private static final Logger LOG = LogManager.getLogger(RestoreJob.class); @@ -211,6 +212,8 @@ public class RestoreJob extends AbstractJob implements GsonPostProcessable { private boolean isCleanPartitions = false; // Whether to restore the data into a temp table, and then replace the origin one. private boolean isAtomicRestore = false; + // Whether to restore the table by replacing the exists but conflicted table. + private boolean isForceReplace = false; // restore properties @SerializedName("prop") @@ -229,7 +232,7 @@ public class RestoreJob extends AbstractJob implements GsonPostProcessable { public RestoreJob(String label, String backupTs, long dbId, String dbName, BackupJobInfo jobInfo, boolean allowLoad, ReplicaAllocation replicaAlloc, long timeoutMs, int metaVersion, boolean reserveReplica, boolean reserveDynamicPartitionEnable, boolean isBeingSynced, boolean isCleanTables, - boolean isCleanPartitions, boolean isAtomicRestore, Env env, long repoId) { + boolean isCleanPartitions, boolean isAtomicRestore, boolean isForceReplace, Env env, long repoId) { super(JobType.RESTORE, label, dbId, dbName, timeoutMs, env, repoId); this.backupTimestamp = backupTs; this.jobInfo = jobInfo; @@ -238,7 +241,8 @@ public class RestoreJob extends AbstractJob implements GsonPostProcessable { this.state = RestoreJobState.PENDING; this.metaVersion = metaVersion; this.reserveReplica = reserveReplica; - // if backup snapshot is come from a cluster with force replication allocation, ignore the origin allocation + // if backup snapshot is come from a cluster with force replication allocation, + // ignore the origin allocation if (jobInfo.isForceReplicationAllocation) { this.reserveReplica = false; } @@ -247,20 +251,26 @@ public class RestoreJob extends AbstractJob implements GsonPostProcessable { this.isCleanTables = isCleanTables; this.isCleanPartitions = isCleanPartitions; this.isAtomicRestore = isAtomicRestore; + if (this.isAtomicRestore) { + this.isForceReplace = isForceReplace; + } properties.put(PROP_RESERVE_REPLICA, String.valueOf(reserveReplica)); properties.put(PROP_RESERVE_DYNAMIC_PARTITION_ENABLE, String.valueOf(reserveDynamicPartitionEnable)); properties.put(PROP_IS_BEING_SYNCED, String.valueOf(isBeingSynced)); properties.put(PROP_CLEAN_TABLES, String.valueOf(isCleanTables)); properties.put(PROP_CLEAN_PARTITIONS, String.valueOf(isCleanPartitions)); properties.put(PROP_ATOMIC_RESTORE, String.valueOf(isAtomicRestore)); + properties.put(PROP_FORCE_REPLACE, String.valueOf(isForceReplace)); } public RestoreJob(String label, String backupTs, long dbId, String dbName, BackupJobInfo jobInfo, boolean allowLoad, ReplicaAllocation replicaAlloc, long timeoutMs, int metaVersion, boolean reserveReplica, boolean reserveDynamicPartitionEnable, boolean isBeingSynced, boolean isCleanTables, - boolean isCleanPartitions, boolean isAtomicRestore, Env env, long repoId, BackupMeta backupMeta) { + boolean isCleanPartitions, boolean isAtomicRestore, boolean isForceReplace, Env env, long repoId, + BackupMeta backupMeta) { this(label, backupTs, dbId, dbName, jobInfo, allowLoad, replicaAlloc, timeoutMs, metaVersion, reserveReplica, - reserveDynamicPartitionEnable, isBeingSynced, isCleanTables, isCleanPartitions, isAtomicRestore, env, + reserveDynamicPartitionEnable, isBeingSynced, isCleanTables, isCleanPartitions, isAtomicRestore, + isForceReplace, env, repoId); this.backupMeta = backupMeta; } @@ -679,6 +689,7 @@ public class RestoreJob extends AbstractJob implements GsonPostProcessable { Table remoteTbl = backupMeta.getTable(tableName); Preconditions.checkNotNull(remoteTbl); Table localTbl = db.getTableNullable(jobInfo.getAliasByOriginNameIfSet(tableName)); + boolean isSchemaChanged = false; if (localTbl != null && localTbl.getType() != TableType.OLAP) { // table already exist, but is not OLAP status = new Status(ErrCode.COMMON_ERROR, @@ -696,8 +707,14 @@ public class RestoreJob extends AbstractJob implements GsonPostProcessable { List<String> intersectPartNames = Lists.newArrayList(); Status st = localOlapTbl.getIntersectPartNamesWith(remoteOlapTbl, intersectPartNames); if (!st.ok()) { - status = st; - return; + if (isForceReplace) { + LOG.info("{}, will force replace, job: {}", + st.getErrMsg(), this); + isSchemaChanged = true; + } else { + status = st; + return; + } } if (LOG.isDebugEnabled()) { LOG.debug("get intersect part names: {}, job: {}", intersectPartNames, this); @@ -707,13 +724,20 @@ public class RestoreJob extends AbstractJob implements GsonPostProcessable { String remoteTblSignature = remoteOlapTbl.getSignature( BackupHandler.SIGNATURE_VERSION, intersectPartNames); if (!localTblSignature.equals(remoteTblSignature)) { - String alias = jobInfo.getAliasByOriginNameIfSet(tableName); - LOG.warn("Table {} already exists but with different schema, " - + "local table: {}, remote table: {}", - alias, localTblSignature, remoteTblSignature); - status = new Status(ErrCode.COMMON_ERROR, "Table " - + alias + " already exist but with different schema"); - return; + if (isForceReplace) { + LOG.info("Table {} already exists but with different schema, will force replace, " + + "local table: {}, remote table: {}", + tableName, localTblSignature, remoteTblSignature); + isSchemaChanged = true; + } else { + String alias = jobInfo.getAliasByOriginNameIfSet(tableName); + LOG.warn("Table {} already exists but with different schema, " + + "local table: {}, remote table: {}", + alias, localTblSignature, remoteTblSignature); + status = new Status(ErrCode.COMMON_ERROR, "Table " + + alias + " already exist but with different schema"); + return; + } } // Table with same name and has same schema. Check partition @@ -735,10 +759,14 @@ public class RestoreJob extends AbstractJob implements GsonPostProcessable { .getPartitionInfo().getItem(backupPartInfo.id); if (!localItem.equals(remoteItem)) { // Same partition name, different range - status = new Status(ErrCode.COMMON_ERROR, "Partition " + partitionName - + " in table " + localTbl.getName() - + " has different partition item with partition in repository"); - return; + if (isForceReplace) { + isSchemaChanged = true; + } else { + status = new Status(ErrCode.COMMON_ERROR, "Partition " + partitionName + + " in table " + localTbl.getName() + + " has different partition item with partition in repository"); + return; + } } } @@ -824,7 +852,7 @@ public class RestoreJob extends AbstractJob implements GsonPostProcessable { // remoteOlapTbl.setName(jobInfo.getAliasByOriginNameIfSet(tblInfo.name)); remoteOlapTbl.setState(allowLoad ? OlapTableState.RESTORE_WITH_LOAD : OlapTableState.RESTORE); - if (isAtomicRestore && localTbl != null) { + if (isAtomicRestore && localTbl != null && !isSchemaChanged) { // bind the backends and base tablets from local tbl. status = bindLocalAndRemoteOlapTableReplicas((OlapTable) localTbl, remoteOlapTbl, tabletBases); if (!status.ok()) { @@ -2411,7 +2439,6 @@ public class RestoreJob extends AbstractJob implements GsonPostProcessable { } private Status atomicReplaceOlapTables(Database db, boolean isReplay) { - assert isAtomicRestore; for (String tableName : jobInfo.backupOlapTableObjects.keySet()) { String originName = jobInfo.getAliasByOriginNameIfSet(tableName); if (Env.isStoredTableNamesLowerCase()) { @@ -2425,13 +2452,16 @@ public class RestoreJob extends AbstractJob implements GsonPostProcessable { try { Table newTbl = db.getTableNullable(aliasName); if (newTbl == null) { - LOG.warn("replace table from {} to {}, but the temp table is not found", aliasName, originName); + LOG.warn("replace table from {} to {}, but the temp table is not found" + " isAtomicRestore: {}", + aliasName, originName, isAtomicRestore); return new Status(ErrCode.COMMON_ERROR, "replace table failed, the temp table " + aliasName + " is not found"); } if (newTbl.getType() != TableType.OLAP) { - LOG.warn("replace table from {} to {}, but the temp table is not OLAP, it type is {}", - aliasName, originName, newTbl.getType()); + LOG.warn( + "replace table from {} to {}, but the temp table is not OLAP, it type is {}" + + " isAtomicRestore: {}", + aliasName, originName, newTbl.getType(), isAtomicRestore); return new Status(ErrCode.COMMON_ERROR, "replace table failed, the temp table " + aliasName + " is not OLAP table, it is " + newTbl.getType()); } @@ -2440,12 +2470,14 @@ public class RestoreJob extends AbstractJob implements GsonPostProcessable { Table originTbl = db.getTableNullable(originName); if (originTbl != null) { if (originTbl.getType() != TableType.OLAP) { - LOG.warn("replace table from {} to {}, but the origin table is not OLAP, it type is {}", - aliasName, originName, originTbl.getType()); + LOG.warn( + "replace table from {} to {}, but the origin table is not OLAP, it type is {}" + + " isAtomicRestore: {}", + aliasName, originName, originTbl.getType(), isAtomicRestore); return new Status(ErrCode.COMMON_ERROR, "replace table failed, the origin table " + originName + " is not OLAP table, it is " + originTbl.getType()); } - originOlapTbl = (OlapTable) originTbl; // save the origin olap table, then drop it. + originOlapTbl = (OlapTable) originTbl; // save the origin olap table, then drop it. } // replace the table. @@ -2460,11 +2492,14 @@ public class RestoreJob extends AbstractJob implements GsonPostProcessable { // set the olap table state to normal immediately for querying newOlapTbl.setState(OlapTableState.NORMAL); - LOG.info("atomic restore replace table {} name to {}, and set state to normal, origin table={}", - newOlapTbl.getId(), originName, originOlapTbl == null ? -1L : originOlapTbl.getId()); + LOG.info( + "restore with replace table {} name to {}, and set state to normal, origin table={}" + + " isAtomicRestore: {}", + newOlapTbl.getId(), originName, originOlapTbl == null ? -1L : originOlapTbl.getId(), + isAtomicRestore); } catch (DdlException e) { - LOG.warn("atomic restore replace table {} name from {} to {}", - newOlapTbl.getId(), aliasName, originName, e); + LOG.warn("restore with replace table {} name from {} to {}, isAtomicRestore: {}", + newOlapTbl.getId(), aliasName, originName, isAtomicRestore, e); return new Status(ErrCode.COMMON_ERROR, "replace table from " + aliasName + " to " + originName + " failed, reason=" + e.getMessage()); } finally { @@ -2475,8 +2510,8 @@ public class RestoreJob extends AbstractJob implements GsonPostProcessable { // The origin table is not used anymore, need to drop all its tablets. originOlapTbl.writeLock(); try { - LOG.info("drop the origin olap table {} by atomic restore. table={}", - originOlapTbl.getName(), originOlapTbl.getId()); + LOG.info("drop the origin olap table {}. table={}" + " isAtomicRestore: {}", + originOlapTbl.getName(), originOlapTbl.getId(), isAtomicRestore); Env.getCurrentEnv().onEraseOlapTable(originOlapTbl, isReplay); } finally { originOlapTbl.writeUnlock(); @@ -2664,6 +2699,7 @@ public class RestoreJob extends AbstractJob implements GsonPostProcessable { isCleanTables = Boolean.parseBoolean(properties.get(PROP_CLEAN_TABLES)); isCleanPartitions = Boolean.parseBoolean(properties.get(PROP_CLEAN_PARTITIONS)); isAtomicRestore = Boolean.parseBoolean(properties.get(PROP_ATOMIC_RESTORE)); + isForceReplace = Boolean.parseBoolean(properties.get(PROP_FORCE_REPLACE)); } @Override @@ -2674,6 +2710,7 @@ public class RestoreJob extends AbstractJob implements GsonPostProcessable { isCleanTables = Boolean.parseBoolean(properties.get(PROP_CLEAN_TABLES)); isCleanPartitions = Boolean.parseBoolean(properties.get(PROP_CLEAN_PARTITIONS)); isAtomicRestore = Boolean.parseBoolean(properties.get(PROP_ATOMIC_RESTORE)); + isForceReplace = Boolean.parseBoolean(properties.get(PROP_FORCE_REPLACE)); } @Override diff --git a/fe/fe-core/src/main/java/org/apache/doris/service/FrontendServiceImpl.java b/fe/fe-core/src/main/java/org/apache/doris/service/FrontendServiceImpl.java index 5b288792adc..5f0b250f593 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/service/FrontendServiceImpl.java +++ b/fe/fe-core/src/main/java/org/apache/doris/service/FrontendServiceImpl.java @@ -3081,6 +3081,9 @@ public class FrontendServiceImpl implements FrontendService.Iface { if (request.isAtomicRestore()) { properties.put(RestoreStmt.PROP_ATOMIC_RESTORE, "true"); } + if (request.isForceReplace()) { + properties.put(RestoreStmt.PROP_FORCE_REPLACE, "true"); + } AbstractBackupTableRefClause restoreTableRefClause = null; if (request.isSetTableRefs()) { diff --git a/fe/fe-core/src/test/java/org/apache/doris/backup/RestoreJobTest.java b/fe/fe-core/src/test/java/org/apache/doris/backup/RestoreJobTest.java index dadfdb632e3..300e46a2b4c 100644 --- a/fe/fe-core/src/test/java/org/apache/doris/backup/RestoreJobTest.java +++ b/fe/fe-core/src/test/java/org/apache/doris/backup/RestoreJobTest.java @@ -256,7 +256,8 @@ public class RestoreJobTest { db.unregisterTable(expectedRestoreTbl.getName()); job = new RestoreJob(label, "2018-01-01 01:01:01", db.getId(), db.getFullName(), jobInfo, false, - new ReplicaAllocation((short) 3), 100000, -1, false, false, false, false, false, false, + new ReplicaAllocation((short) 3), 100000, -1, false, false, + false, false, false, false, false, env, repo.getId()); List<Table> tbls = Lists.newArrayList(); diff --git a/gensrc/thrift/FrontendService.thrift b/gensrc/thrift/FrontendService.thrift index 8f79b2b98ac..f27b8bff9a4 100644 --- a/gensrc/thrift/FrontendService.thrift +++ b/gensrc/thrift/FrontendService.thrift @@ -1415,6 +1415,7 @@ struct TRestoreSnapshotRequest { 14: optional bool clean_partitions 15: optional bool atomic_restore 16: optional bool compressed; + 17: optional bool force_replace } struct TRestoreSnapshotResult { diff --git a/regression-test/suites/backup_restore/test_backup_restore_force_replace_diff_column.groovy b/regression-test/suites/backup_restore/test_backup_restore_force_replace_diff_column.groovy new file mode 100644 index 00000000000..79771d218d8 --- /dev/null +++ b/regression-test/suites/backup_restore/test_backup_restore_force_replace_diff_column.groovy @@ -0,0 +1,96 @@ +// 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_backup_restore_force_replace_diff_column", "backup_restore") { + String suiteName = "test_backup_restore_force_replace_diff_column" + String dbName = "${suiteName}_db_0" + String repoName = "repo_" + UUID.randomUUID().toString().replace("-", "") + String snapshotName = "${suiteName}_snapshot_" + System.currentTimeMillis() + String tableNamePrefix = "${suiteName}_tables" + String tableName = "${tableNamePrefix}_0" + + def syncer = getSyncer() + syncer.createS3Repository(repoName) + sql "DROP DATABASE IF EXISTS ${dbName}" + sql "CREATE DATABASE IF NOT EXISTS ${dbName}" + + sql "DROP TABLE IF EXISTS ${dbName}.${tableName}" + sql """ + CREATE TABLE ${dbName}.${tableName} ( + `id` LARGEINT NOT NULL, + `count` LARGEINT SUM DEFAULT "0" + ) + PARTITION BY RANGE(`id`) + ( + ) + DISTRIBUTED BY HASH(`id`) BUCKETS 2 + PROPERTIES + ( + "replication_num" = "1" + ) + """ + + sql """ + BACKUP SNAPSHOT ${dbName}.${snapshotName} + TO `${repoName}` + ON ( + ${tableName} + ) + """ + + syncer.waitSnapshotFinish(dbName) + + def snapshot = syncer.getSnapshotTimestamp(repoName, snapshotName) + assertTrue(snapshot != null) + + sql "DROP TABLE ${dbName}.${tableName}" + + sql """ + CREATE TABLE ${dbName}.${tableName} ( + `id` LARGEINT NOT NULL, + `count` LARGEINT DEFAULT "0", + `desc` VARCHAR(20) DEFAULT "" + ) + PARTITION BY RANGE(`id`) + ( + ) + DISTRIBUTED BY HASH(`id`) BUCKETS 2 + PROPERTIES + ( + "replication_num" = "1" + ) + """ + + sql """ + RESTORE SNAPSHOT ${dbName}.${snapshotName} + FROM `${repoName}` + PROPERTIES + ( + "backup_timestamp" = "${snapshot}", + "reserve_replica" = "true", + "atomic_restore" = "true", + "force_replace" = "true" + ) + """ + + syncer.waitAllRestoreFinish(dbName) + + sql "sync" + def desc_res = sql "desc ${dbName}.${tableName}" + assertEquals(desc_res.size(), 2) +} + diff --git a/regression-test/suites/backup_restore/test_backup_restore_force_replace_diff_part_type.groovy b/regression-test/suites/backup_restore/test_backup_restore_force_replace_diff_part_type.groovy new file mode 100644 index 00000000000..f42932de1cd --- /dev/null +++ b/regression-test/suites/backup_restore/test_backup_restore_force_replace_diff_part_type.groovy @@ -0,0 +1,97 @@ +// 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_backup_restore_force_replace_diff_part_type", "backup_restore") { + String suiteName = "test_backup_restore_force_replace_diff_part_type" + String dbName = "${suiteName}_db_0" + String repoName = "repo_" + UUID.randomUUID().toString().replace("-", "") + String snapshotName = "${suiteName}_snapshot_" + System.currentTimeMillis() + String tableNamePrefix = "${suiteName}_tables" + String tableName = "${tableNamePrefix}_0" + + def syncer = getSyncer() + syncer.createS3Repository(repoName) + sql "DROP DATABASE IF EXISTS ${dbName}" + sql "CREATE DATABASE IF NOT EXISTS ${dbName}" + + sql "DROP TABLE IF EXISTS ${dbName}.${tableName}" + sql """ + CREATE TABLE ${dbName}.${tableName} ( + `id` LARGEINT NOT NULL, + `count` LARGEINT SUM DEFAULT "0" + ) + AGGREGATE KEY(`id`) + PARTITION BY LIST(`id`) + ( + ) + DISTRIBUTED BY HASH(`id`) BUCKETS 2 + PROPERTIES + ( + "replication_num" = "1" + ) + """ + + sql """ + BACKUP SNAPSHOT ${dbName}.${snapshotName} + TO `${repoName}` + ON ( + ${tableName} + ) + """ + + syncer.waitSnapshotFinish(dbName) + + def snapshot = syncer.getSnapshotTimestamp(repoName, snapshotName) + assertTrue(snapshot != null) + + sql "DROP TABLE ${dbName}.${tableName}" + + sql """ + CREATE TABLE ${dbName}.${tableName} ( + `id` LARGEINT NOT NULL, + `count` LARGEINT DEFAULT "0" + ) + AGGREGATE KEY(`id`, `count`) + PARTITION BY RANGE(`id`) + ( + ) + DISTRIBUTED BY HASH(`id`) BUCKETS 2 + PROPERTIES + ( + "replication_num" = "1" + ) + """ + + sql """ + RESTORE SNAPSHOT ${dbName}.${snapshotName} + FROM `${repoName}` + PROPERTIES + ( + "backup_timestamp" = "${snapshot}", + "reserve_replica" = "true", + "atomic_restore" = "true", + "force_replace" = "true" + ) + """ + + syncer.waitAllRestoreFinish(dbName) + + sql "sync" + def show_table = sql "show create table ${dbName}.${tableName}" + assertTrue(show_table[0][1].contains("PARTITION BY LIST (`id`)")) +} + diff --git a/regression-test/suites/backup_restore/test_backup_restore_force_replace_diff_part_val.groovy b/regression-test/suites/backup_restore/test_backup_restore_force_replace_diff_part_val.groovy new file mode 100644 index 00000000000..38b5c10b628 --- /dev/null +++ b/regression-test/suites/backup_restore/test_backup_restore_force_replace_diff_part_val.groovy @@ -0,0 +1,101 @@ +// 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_backup_restore_force_replace_diff_part_val", "backup_restore") { + String suiteName = "test_backup_restore_force_replace_diff_part_val" + String dbName = "${suiteName}_db_0" + String repoName = "repo_" + UUID.randomUUID().toString().replace("-", "") + String snapshotName = "${suiteName}_snapshot_" + System.currentTimeMillis() + String tableNamePrefix = "${suiteName}_tables" + String tableName = "${tableNamePrefix}_0" + + def syncer = getSyncer() + syncer.createS3Repository(repoName) + sql "CREATE DATABASE IF NOT EXISTS ${dbName}" + + sql "DROP TABLE IF EXISTS ${dbName}.${tableName}" + sql """ + CREATE TABLE ${dbName}.${tableName} ( + `id` LARGEINT NOT NULL, + `count` LARGEINT SUM DEFAULT "0" + ) + AGGREGATE KEY(`id`) + PARTITION BY RANGE(`id`) + ( + PARTITION p0 VALUES LESS THAN ("10") + ) + DISTRIBUTED BY HASH(`id`) BUCKETS 2 + PROPERTIES + ( + "replication_num" = "1" + ) + """ + + sql """ + BACKUP SNAPSHOT ${dbName}.${snapshotName} + TO `${repoName}` + ON ( + ${tableName} + ) + """ + + syncer.waitSnapshotFinish(dbName) + + def snapshot = syncer.getSnapshotTimestamp(repoName, snapshotName) + assertTrue(snapshot != null) + + sql "DROP TABLE ${dbName}.${tableName}" + + sql """ + CREATE TABLE ${dbName}.${tableName} ( + `id` LARGEINT NOT NULL, + `count` LARGEINT DEFAULT "0" + ) + AGGREGATE KEY(`id`, `count`) + PARTITION BY RANGE(`id`, `count`) + ( + PARTITION p0 VALUES LESS THAN ("100") + ) + DISTRIBUTED BY HASH(`id`) BUCKETS 2 + PROPERTIES + ( + "replication_num" = "1" + ) + """ + + sql """ + RESTORE SNAPSHOT ${dbName}.${snapshotName} + FROM `${repoName}` + PROPERTIES + ( + "backup_timestamp" = "${snapshot}", + "reserve_replica" = "true", + "atomic_restore" = "true", + "force_replace" = "true" + ) + """ + + syncer.waitAllRestoreFinish(dbName) + + sql "sync" + def show_partitions = sql_return_maparray "SHOW PARTITIONS FROM ${dbName}.${tableName} where PartitionName = 'p0'" + + logger.info(show_partitions.Range) + + assertTrue(show_partitions.Range.contains("[types: [LARGEINT]; keys: [-170141183460469231731687303715884105728]; ..types: [LARGEINT]; keys: [10]; )")) +} + --------------------------------------------------------------------- To unsubscribe, e-mail: commits-unsubscr...@doris.apache.org For additional commands, e-mail: commits-h...@doris.apache.org