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 60a0d1ca8ff [feature](cloud) Support rename compute group sql (#46221) 60a0d1ca8ff is described below commit 60a0d1ca8ff9c37c59c444bc27ae7d00f3586449 Author: deardeng <deng...@selectdb.com> AuthorDate: Tue Jan 14 23:04:17 2025 +0800 [feature](cloud) Support rename compute group sql (#46221) --- cloud/src/meta-service/meta_service_resource.cpp | 12 +- cloud/src/resource-manager/resource_manager.cpp | 30 ++- cloud/src/resource-manager/resource_manager.h | 4 +- cloud/test/mock_resource_manager.h | 4 +- .../antlr4/org/apache/doris/nereids/DorisParser.g4 | 1 + .../doris/cloud/catalog/CloudClusterChecker.java | 18 +- .../org/apache/doris/cloud/catalog/CloudEnv.java | 4 + .../doris/cloud/system/CloudSystemInfoService.java | 65 ++++++- .../doris/nereids/parser/LogicalPlanBuilder.java | 7 + .../apache/doris/nereids/trees/plans/PlanType.java | 3 +- .../AlterSystemRenameComputeGroupCommand.java | 85 ++++++++ gensrc/proto/cloud.proto | 2 + .../node_mgr/test_rename_compute_group.groovy | 214 +++++++++++++++++++++ 13 files changed, 430 insertions(+), 19 deletions(-) diff --git a/cloud/src/meta-service/meta_service_resource.cpp b/cloud/src/meta-service/meta_service_resource.cpp index 23dc9d0b40c..529daa72c1a 100644 --- a/cloud/src/meta-service/meta_service_resource.cpp +++ b/cloud/src/meta-service/meta_service_resource.cpp @@ -2203,6 +2203,13 @@ void MetaServiceImpl::alter_cluster(google::protobuf::RpcController* controller, } } break; case AlterClusterRequest::RENAME_CLUSTER: { + // SQL mode, cluster cluster name eq empty cluster name, need drop empty cluster first. + // but in http api, cloud control will drop empty cluster + bool replace_if_existing_empty_target_cluster = + request->has_replace_if_existing_empty_target_cluster() + ? request->replace_if_existing_empty_target_cluster() + : false; + msg = resource_mgr_->update_cluster( instance_id, cluster, [&](const ClusterPB& i) { return i.cluster_id() == cluster.cluster.cluster_id(); }, @@ -2212,7 +2219,7 @@ void MetaServiceImpl::alter_cluster(google::protobuf::RpcController* controller, LOG(INFO) << "cluster.cluster.cluster_name(): " << cluster.cluster.cluster_name(); for (auto itt : cluster_names) { - LOG(INFO) << "itt : " << itt; + LOG(INFO) << "instance's cluster name : " << itt; } if (it != cluster_names.end()) { code = MetaServiceCode::INVALID_ARGUMENT; @@ -2232,7 +2239,8 @@ void MetaServiceImpl::alter_cluster(google::protobuf::RpcController* controller, } c.set_cluster_name(cluster.cluster.cluster_name()); return msg; - }); + }, + replace_if_existing_empty_target_cluster); } break; case AlterClusterRequest::UPDATE_CLUSTER_ENDPOINT: { msg = resource_mgr_->update_cluster( diff --git a/cloud/src/resource-manager/resource_manager.cpp b/cloud/src/resource-manager/resource_manager.cpp index 3addfecdb85..827ad318502 100644 --- a/cloud/src/resource-manager/resource_manager.cpp +++ b/cloud/src/resource-manager/resource_manager.cpp @@ -577,7 +577,8 @@ std::pair<MetaServiceCode, std::string> ResourceManager::drop_cluster( std::string ResourceManager::update_cluster( const std::string& instance_id, const ClusterInfo& cluster, std::function<bool(const ClusterPB&)> filter, - std::function<std::string(ClusterPB&, std::set<std::string>& cluster_names)> action) { + std::function<std::string(ClusterPB&, std::set<std::string>& cluster_names)> action, + bool replace_if_existing_empty_target_cluster) { std::stringstream ss; std::string msg; @@ -643,6 +644,33 @@ std::string ResourceManager::update_cluster( auto& clusters = const_cast<std::decay_t<decltype(instance.clusters())>&>(instance.clusters()); + // check cluster_name is empty cluster, if empty and replace_if_existing_empty_target_cluster == true, drop it + if (replace_if_existing_empty_target_cluster) { + auto it = cluster_names.find(cluster_name); + if (it != cluster_names.end()) { + // found it, if it's an empty cluster, drop it from instance + int idx = -1; + for (auto& cluster : instance.clusters()) { + idx++; + if (cluster.cluster_name() == cluster_name) { + // Check if cluster is empty (has no nodes) + if (cluster.nodes_size() == 0) { + // Remove empty cluster from instance + auto& clusters = const_cast<std::decay_t<decltype(instance.clusters())>&>( + instance.clusters()); + clusters.DeleteSubrange(idx, 1); + // Remove cluster name from set + cluster_names.erase(cluster_name); + LOG(INFO) << "remove empty cluster due to it is the target of a " + "rename_cluster, cluster_name=" + << cluster_name; + } + break; + } + } + } + } + // do update ClusterPB original = clusters[idx]; msg = action(clusters[idx], cluster_names); diff --git a/cloud/src/resource-manager/resource_manager.h b/cloud/src/resource-manager/resource_manager.h index 9e6f4548d24..21f09d34a37 100644 --- a/cloud/src/resource-manager/resource_manager.h +++ b/cloud/src/resource-manager/resource_manager.h @@ -88,13 +88,15 @@ public: * * @param cluster cluster to update, only cluster name and cluster id are concered * @param action update operation code snippet + * @param replace_if_existing_empty_target_cluster, find cluster.cluster_name is a empty cluster(no node), drop it * @filter filter condition * @return empty string for success, otherwise failure reason returned */ virtual std::string update_cluster( const std::string& instance_id, const ClusterInfo& cluster, std::function<bool(const ClusterPB&)> filter, - std::function<std::string(ClusterPB&, std::set<std::string>& cluster_names)> action); + std::function<std::string(ClusterPB&, std::set<std::string>& cluster_names)> action, + bool replace_if_existing_empty_target_cluster = false); /** * Get instance from underlying storage with given transaction. diff --git a/cloud/test/mock_resource_manager.h b/cloud/test/mock_resource_manager.h index 748947cb46a..25b0d5fbb4b 100644 --- a/cloud/test/mock_resource_manager.h +++ b/cloud/test/mock_resource_manager.h @@ -59,8 +59,8 @@ public: std::string update_cluster( const std::string& instance_id, const ClusterInfo& cluster, std::function<bool(const ClusterPB&)> filter, - std::function<std::string(ClusterPB&, std::set<std::string>& cluster_names)> action) - override { + std::function<std::string(ClusterPB&, std::set<std::string>& cluster_names)> action, + bool replace_if_existing_empty_target_cluster) override { return ""; } diff --git a/fe/fe-core/src/main/antlr4/org/apache/doris/nereids/DorisParser.g4 b/fe/fe-core/src/main/antlr4/org/apache/doris/nereids/DorisParser.g4 index 5235c040219..8c21461f1e6 100644 --- a/fe/fe-core/src/main/antlr4/org/apache/doris/nereids/DorisParser.g4 +++ b/fe/fe-core/src/main/antlr4/org/apache/doris/nereids/DorisParser.g4 @@ -224,6 +224,7 @@ supportedAlterStatement dropRollupClause (COMMA dropRollupClause)* #alterTableDropRollup | ALTER TABLE name=multipartIdentifier SET LEFT_PAREN propertyItemList RIGHT_PAREN #alterTableProperties + | ALTER SYSTEM RENAME COMPUTE GROUP name=identifier newName=identifier #alterSystemRenameComputeGroup ; supportedDropStatement diff --git a/fe/fe-core/src/main/java/org/apache/doris/cloud/catalog/CloudClusterChecker.java b/fe/fe-core/src/main/java/org/apache/doris/cloud/catalog/CloudClusterChecker.java index 9468c8acecd..b6756fb5cdf 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/cloud/catalog/CloudClusterChecker.java +++ b/fe/fe-core/src/main/java/org/apache/doris/cloud/catalog/CloudClusterChecker.java @@ -52,6 +52,8 @@ public class CloudClusterChecker extends MasterDaemon { private CloudSystemInfoService cloudSystemInfoService; + private final Object checkLock = new Object(); + boolean isUpdateCloudUniqueId = false; public CloudClusterChecker(CloudSystemInfoService cloudSystemInfoService) { @@ -321,9 +323,11 @@ public class CloudClusterChecker extends MasterDaemon { @Override protected void runAfterCatalogReady() { - checkCloudBackends(); - updateCloudMetrics(); - checkCloudFes(); + synchronized (checkLock) { + checkCloudBackends(); + updateCloudMetrics(); + checkCloudFes(); + } } private void checkFeNodesMapValid() { @@ -545,4 +549,12 @@ public class CloudClusterChecker extends MasterDaemon { MetricRepo.updateClusterBackendAliveTotal(entry.getKey(), entry.getValue(), aliveNum); } } + + public void checkNow() { + if (Env.getCurrentEnv().isMaster()) { + synchronized (checkLock) { + runAfterCatalogReady(); + } + } + } } diff --git a/fe/fe-core/src/main/java/org/apache/doris/cloud/catalog/CloudEnv.java b/fe/fe-core/src/main/java/org/apache/doris/cloud/catalog/CloudEnv.java index 7aeb35ede68..190cb457a94 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/cloud/catalog/CloudEnv.java +++ b/fe/fe-core/src/main/java/org/apache/doris/cloud/catalog/CloudEnv.java @@ -94,6 +94,10 @@ public class CloudEnv extends Env { return this.upgradeMgr; } + public CloudClusterChecker getCloudClusterChecker() { + return this.cloudClusterCheck; + } + public String getCloudInstanceId() { return cloudInstanceId; } diff --git a/fe/fe-core/src/main/java/org/apache/doris/cloud/system/CloudSystemInfoService.java b/fe/fe-core/src/main/java/org/apache/doris/cloud/system/CloudSystemInfoService.java index 71260c51f23..e366efb6595 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/cloud/system/CloudSystemInfoService.java +++ b/fe/fe-core/src/main/java/org/apache/doris/cloud/system/CloudSystemInfoService.java @@ -327,9 +327,9 @@ public class CloudSystemInfoService extends SystemInfoService { throw new DdlException("unable to alter backends due to empty cloud_instance_id"); } // Issue rpc to meta to alter node, then fe master would add this node to its frontends - Cloud.ClusterPB clusterPB = Cloud.ClusterPB.newBuilder() + ClusterPB clusterPB = ClusterPB.newBuilder() .setClusterId(computeGroupId) - .setType(Cloud.ClusterPB.Type.COMPUTE) + .setType(ClusterPB.Type.COMPUTE) .build(); for (HostInfo hostInfo : hostInfos) { @@ -847,10 +847,10 @@ public class CloudSystemInfoService extends SystemInfoService { .setCtime(System.currentTimeMillis() / 1000) .build(); - Cloud.ClusterPB clusterPB = Cloud.ClusterPB.newBuilder() + ClusterPB clusterPB = ClusterPB.newBuilder() .setClusterId(Config.cloud_sql_server_cluster_id) .setClusterName(Config.cloud_sql_server_cluster_name) - .setType(Cloud.ClusterPB.Type.SQL) + .setType(ClusterPB.Type.SQL) .addNodes(nodeInfoPB) .build(); @@ -888,13 +888,13 @@ public class CloudSystemInfoService extends SystemInfoService { private String tryCreateComputeGroup(String clusterName, String computeGroupId) throws UserException { if (Strings.isNullOrEmpty(((CloudEnv) Env.getCurrentEnv()).getCloudInstanceId())) { - throw new DdlException("unable to create compute group due to empty cluster_id"); + throw new DdlException("unable to create compute group due to empty cloud_instance_id"); } - Cloud.ClusterPB clusterPB = Cloud.ClusterPB.newBuilder() + ClusterPB clusterPB = ClusterPB.newBuilder() .setClusterId(computeGroupId) .setClusterName(clusterName) - .setType(Cloud.ClusterPB.Type.COMPUTE) + .setType(ClusterPB.Type.COMPUTE) .build(); Cloud.AlterClusterRequest request = Cloud.AlterClusterRequest.newBuilder() @@ -920,7 +920,7 @@ public class CloudSystemInfoService extends SystemInfoService { Cloud.GetClusterResponse clusterResponse = getCloudCluster(clusterName, "", ""); if (clusterResponse.getStatus().getCode() == Cloud.MetaServiceCode.OK) { if (clusterResponse.getClusterCount() > 0) { - Cloud.ClusterPB cluster = clusterResponse.getCluster(0); + ClusterPB cluster = clusterResponse.getCluster(0); return cluster.getClusterId(); } else { throw new UserException("Cluster information not found in the response"); @@ -1057,7 +1057,7 @@ public class CloudSystemInfoService extends SystemInfoService { builder.setCloudUniqueId(Config.cloud_unique_id); builder.setOp(Cloud.AlterClusterRequest.Operation.SET_CLUSTER_STATUS); - Cloud.ClusterPB.Builder clusterBuilder = Cloud.ClusterPB.newBuilder(); + ClusterPB.Builder clusterBuilder = ClusterPB.newBuilder(); clusterBuilder.setClusterId(getCloudClusterIdByName(clusterName)); clusterBuilder.setClusterStatus(Cloud.ClusterStatus.TO_RESUME); builder.setCluster(clusterBuilder); @@ -1161,4 +1161,51 @@ public class CloudSystemInfoService extends SystemInfoService { throw new IOException("Failed to get instance info"); } } + + public void renameComputeGroup(String originalName, String newGroupName) throws UserException { + String cloudInstanceId = ((CloudEnv) Env.getCurrentEnv()).getCloudInstanceId(); + if (Strings.isNullOrEmpty(cloudInstanceId)) { + throw new DdlException("unable to rename compute group due to empty cloud_instance_id"); + } + String originalComputeGroupId = ((CloudSystemInfoService) Env.getCurrentSystemInfo()) + .getCloudClusterIdByName(originalName); + if (Strings.isNullOrEmpty(originalComputeGroupId)) { + LOG.info("rename original compute group {} not found, unable to rename", originalName); + throw new DdlException("compute group '" + originalName + "' not found, unable to rename"); + } + // check newGroupName has existed + if (((CloudSystemInfoService) Env.getCurrentSystemInfo()).getCloudClusterNames().contains(newGroupName)) { + LOG.info("rename new compute group {} has existed in instance, unable to rename", newGroupName); + throw new DdlException("compute group '" + newGroupName + "' has existed in warehouse, unable to rename"); + } + + ClusterPB clusterPB = ClusterPB.newBuilder() + .setClusterId(originalComputeGroupId) + .setClusterName(newGroupName) + .setType(ClusterPB.Type.COMPUTE) + .build(); + + Cloud.AlterClusterRequest request = Cloud.AlterClusterRequest.newBuilder() + .setInstanceId(((CloudEnv) Env.getCurrentEnv()).getCloudInstanceId()) + .setOp(Cloud.AlterClusterRequest.Operation.RENAME_CLUSTER) + .setReplaceIfExistingEmptyTargetCluster(true) + .setCluster(clusterPB) + .build(); + + + Cloud.AlterClusterResponse response = null; + try { + response = MetaServiceProxy.getInstance().alterCluster(request); + if (response.getStatus().getCode() != Cloud.MetaServiceCode.OK) { + LOG.warn("alter rename compute group not ok, response: {}", response); + throw new UserException("failed to rename compute group errorCode: " + response.getStatus().getCode() + + " msg: " + response.getStatus().getMsg() + " may be you can try later"); + } + } catch (RpcException e) { + LOG.warn("alter rename compute group rpc exception"); + throw new UserException("failed to alter rename compute group", e); + } finally { + LOG.info("alter rename compute group, request: {}, response: {}", request, response); + } + } } diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/parser/LogicalPlanBuilder.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/parser/LogicalPlanBuilder.java index 5880fad3967..476ad72285c 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/parser/LogicalPlanBuilder.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/parser/LogicalPlanBuilder.java @@ -77,6 +77,7 @@ import org.apache.doris.nereids.DorisParser.AlterMultiPartitionClauseContext; import org.apache.doris.nereids.DorisParser.AlterRoleContext; import org.apache.doris.nereids.DorisParser.AlterSqlBlockRuleContext; import org.apache.doris.nereids.DorisParser.AlterStorageVaultContext; +import org.apache.doris.nereids.DorisParser.AlterSystemRenameComputeGroupContext; import org.apache.doris.nereids.DorisParser.AlterTableAddRollupContext; import org.apache.doris.nereids.DorisParser.AlterTableClauseContext; import org.apache.doris.nereids.DorisParser.AlterTableContext; @@ -496,6 +497,7 @@ import org.apache.doris.nereids.trees.plans.commands.AlterMTMVCommand; import org.apache.doris.nereids.trees.plans.commands.AlterRoleCommand; import org.apache.doris.nereids.trees.plans.commands.AlterSqlBlockRuleCommand; import org.apache.doris.nereids.trees.plans.commands.AlterStorageVaultCommand; +import org.apache.doris.nereids.trees.plans.commands.AlterSystemRenameComputeGroupCommand; import org.apache.doris.nereids.trees.plans.commands.AlterTableCommand; import org.apache.doris.nereids.trees.plans.commands.AlterViewCommand; import org.apache.doris.nereids.trees.plans.commands.AlterWorkloadGroupCommand; @@ -1301,6 +1303,11 @@ public class LogicalPlanBuilder extends DorisParserBaseVisitor<Object> { return new AlterStorageVaultCommand(vaultName, properties); } + @Override + public LogicalPlan visitAlterSystemRenameComputeGroup(AlterSystemRenameComputeGroupContext ctx) { + return new AlterSystemRenameComputeGroupCommand(ctx.name.getText(), ctx.newName.getText()); + } + @Override public LogicalPlan visitShowConstraint(ShowConstraintContext ctx) { List<String> parts = visitMultipartIdentifier(ctx.table); diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/PlanType.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/PlanType.java index 97951ccafea..cd1789b687e 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/PlanType.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/PlanType.java @@ -280,5 +280,6 @@ public enum PlanType { SHOW_QUERY_PROFILE_COMMAND, SWITCH_COMMAND, HELP_COMMAND, - USE_COMMAND + USE_COMMAND, + ALTER_SYSTEM_RENAME_COMPUTE_GROUP } diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/commands/AlterSystemRenameComputeGroupCommand.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/commands/AlterSystemRenameComputeGroupCommand.java new file mode 100644 index 00000000000..9c2cf9b2d2a --- /dev/null +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/commands/AlterSystemRenameComputeGroupCommand.java @@ -0,0 +1,85 @@ +// 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. + +package org.apache.doris.nereids.trees.plans.commands; + +import org.apache.doris.catalog.Env; +import org.apache.doris.cloud.catalog.CloudEnv; +import org.apache.doris.cloud.system.CloudSystemInfoService; +import org.apache.doris.common.AnalysisException; +import org.apache.doris.common.DdlException; +import org.apache.doris.common.ErrorCode; +import org.apache.doris.mysql.privilege.PrivPredicate; +import org.apache.doris.nereids.trees.plans.PlanType; +import org.apache.doris.nereids.trees.plans.visitor.PlanVisitor; +import org.apache.doris.qe.ConnectContext; +import org.apache.doris.qe.StmtExecutor; + +import com.google.common.base.Strings; + +/** + * Alter System Rename Compute Group + */ +public class AlterSystemRenameComputeGroupCommand extends Command implements ForwardWithSync { + private final String originalName; + private final String newName; + + public AlterSystemRenameComputeGroupCommand(String originalName, String newName) { + super(PlanType.ALTER_SYSTEM_RENAME_COMPUTE_GROUP); + this.originalName = originalName; + this.newName = newName; + } + + private void validate() throws AnalysisException { + // check admin or root auth, can rename + if (!Env.getCurrentEnv().getAccessManager() + .checkGlobalPriv(ConnectContext.get(), PrivPredicate.ADMIN_OR_NODE)) { + String message = ErrorCode.ERR_SPECIFIC_ACCESS_DENIED_ERROR.formatErrorMsg( + PrivPredicate.ADMIN_OR_NODE.getPrivs().toString()); + throw new org.apache.doris.nereids.exceptions.AnalysisException(message); + } + if (Strings.isNullOrEmpty(originalName) || Strings.isNullOrEmpty(newName)) { + throw new AnalysisException("rename group requires non-empty or non-empty name"); + } + if (originalName.equals(newName)) { + throw new AnalysisException("rename compute group original name eq new name"); + } + } + + @Override + public void run(ConnectContext ctx, StmtExecutor executor) throws Exception { + validate(); + doRun(ctx); + } + + @Override + public <R, C> R accept(PlanVisitor<R, C> visitor, C context) { + return visitor.visitCommand(this, context); + } + + private void doRun(ConnectContext ctx) throws Exception { + try { + // 1. send rename rpc to ms + ((CloudSystemInfoService) Env.getCurrentSystemInfo()).renameComputeGroup(this.originalName, this.newName); + // 2. if 1 not throw exception, refresh cloud cluster + // if not do 2, will wait 10s to get new name + ((CloudEnv) Env.getCurrentEnv()).getCloudClusterChecker().checkNow(); + } catch (Exception e) { + throw new DdlException(e.getMessage()); + } + } +} diff --git a/gensrc/proto/cloud.proto b/gensrc/proto/cloud.proto index 58510c2f138..d82d88d169d 100644 --- a/gensrc/proto/cloud.proto +++ b/gensrc/proto/cloud.proto @@ -1095,6 +1095,8 @@ message AlterClusterRequest { optional string cloud_unique_id = 2; // For auth optional ClusterPB cluster = 3; optional Operation op = 4; + // for SQL mode rename cluster, rename to cluster name eq instance empty cluster name, need drop empty cluster + optional bool replace_if_existing_empty_target_cluster = 5; } message AlterClusterResponse { diff --git a/regression-test/suites/cloud_p0/node_mgr/test_rename_compute_group.groovy b/regression-test/suites/cloud_p0/node_mgr/test_rename_compute_group.groovy new file mode 100644 index 00000000000..35c90c2713d --- /dev/null +++ b/regression-test/suites/cloud_p0/node_mgr/test_rename_compute_group.groovy @@ -0,0 +1,214 @@ +// 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. + +import org.apache.doris.regression.suite.ClusterOptions +import groovy.json.JsonSlurper +import org.awaitility.Awaitility; +import static java.util.concurrent.TimeUnit.SECONDS; + +suite('test_rename_compute_group', 'docker, p0') { + if (!isCloudMode()) { + return; + } + def options = new ClusterOptions() + options.feConfigs += [ + 'sys_log_verbose_modules=org', + 'heartbeat_interval_second=1' + ] + options.setFeNum(2) + options.setBeNum(3) + options.cloudMode = true + options.connectToFollower = true + + def user1 = "test_has_admin_auth_user" + def user2 = "test_no_admin_auth_user" + def table = "test_rename_compute_group_table" + + def get_instance_api = { msHttpPort, instance_id, check_func -> + httpTest { + op "get" + endpoint msHttpPort + uri "/MetaService/http/get_instance?token=${token}&instance_id=${instance_id}" + check check_func + } + } + def findToDropUniqueId = { clusterId, hostIP, metaServices -> + ret = get_instance(metaServices) + def toDropCluster = ret.clusters.find { + it.cluster_id.contains(clusterId) + } + log.info("toDropCluster: {}", toDropCluster) + def toDropNode = toDropCluster.nodes.find { + it.ip.contains(hostIP) + } + log.info("toDropNode: {}", toDropNode) + assertNotNull(toDropCluster) + assertNotNull(toDropNode) + toDropNode.cloud_unique_id + } + + docker(options) { + def clusterName = "newcluster1" + // 添加一个新的cluster add_new_cluster + cluster.addBackend(1, clusterName) + def result = sql """SHOW COMPUTE GROUPS"""; + assertEquals(2, result.size()) + + sql """CREATE USER $user1 IDENTIFIED BY 'Cloud123456' DEFAULT ROLE 'admin';""" + sql """CREATE USER $user2 IDENTIFIED BY 'Cloud123456';""" + // no cluster auth + sql """GRANT SELECT_PRIV ON *.*.* TO ${user2}""" + sql """CREATE TABLE $table ( + `k1` int(11) NULL, + `k2` int(11) NULL + ) + DUPLICATE KEY(`k1`, `k2`) + COMMENT 'OLAP' + DISTRIBUTED BY HASH(`k1`) BUCKETS 1 + PROPERTIES ( + "replication_num"="1" + ); + """ + + // 1. test original compute group not exist in warehouse + try { + sql """ALTER SYSTEM RENAME COMPUTE GROUP notExistComputeGroup compute_cluster;""" + } catch (Exception e) { + logger.info("exception: {}", e.getMessage()) + assertTrue(e.getMessage().contains("compute group 'notExistComputeGroup' not found, unable to rename")) + } + + // 2. test target compute group eq original compute group + try { + sql """ALTER SYSTEM RENAME COMPUTE GROUP compute_cluster compute_cluster;""" + } catch (Exception e) { + logger.info("exception: {}", e.getMessage()) + assertTrue(e.getMessage().contains("rename compute group original name eq new name")) + } + + // 3. test target compute group exist in warehouse + try { + sql """ALTER SYSTEM RENAME COMPUTE GROUP compute_cluster newcluster1;""" + } catch (Exception e) { + logger.info("exception: {}", e.getMessage()) + assertTrue(e.getMessage().contains("compute group 'newcluster1' has existed in warehouse, unable to rename")) + } + // 4. test admin user can rename compute group + connectInDocker(user = user1, password = 'Cloud123456') { + sql """ALTER SYSTEM RENAME COMPUTE GROUP compute_cluster compute_cluster1;""" + sql """sync""" + result = sql_return_maparray """SHOW COMPUTE GROUPS;""" + log.info("show compute group {}", result) + + assertTrue(result.stream().anyMatch(cluster -> cluster.Name == "compute_cluster1")) + assertFalse(result.stream().anyMatch(cluster -> cluster.Name == "compute_cluster")) + // use old compute group name + try { + sql """ use @compute_cluster""" + } catch (Exception e) { + logger.info("exception: {}", e.getMessage()) + assertTrue(e.getMessage().contains("Compute group (aka. Cloud cluster) compute_cluster not exist")) + } + + sql """use @compute_cluster1""" + + // insert succ + sql """ + insert into $table values (1, 1) + """ + + result = sql """ + select count(*) from $table + """ + logger.info("select result {}", result) + assertEquals(1, result[0][0]) + } + + // 5. test non admin user can't rename compute group + connectInDocker(user = user2, password = 'Cloud123456') { + try { + sql """ALTER SYSTEM RENAME COMPUTE GROUP compute_cluster1 compute_cluster2;""" + } catch (Exception e ) { + logger.info("exception: {}", e.getMessage()) + assertTrue(e.getMessage().contains("Access denied; you need (at least one of) the (Node_priv,Admin_priv) privilege(s) for this operation")) + } + } + // 6. test target compute group is empty (no node), can succ, and old empty compute group will be drop + // 调用http api 将add_new_cluster 下掉 + def tag = getCloudBeTagByName(clusterName) + logger.info("tag = {}", tag) + + def jsonSlurper = new JsonSlurper() + def jsonObject = jsonSlurper.parseText(tag) + def cloudClusterId = jsonObject.compute_group_id + def ms = cluster.getAllMetaservices().get(0) + + // tag = {"cloud_unique_id" : "compute_node_4", "compute_group_status" : "NORMAL", "private_endpoint" : "", "compute_group_name" : "newcluster1", "location" : "default", "public_endpoint" : "", "compute_group_id" : "newcluster1_id"} + def toDropIP = cluster.getBeByIndex(4).host + toDropUniqueId = findToDropUniqueId.call(cloudClusterId, toDropIP, ms) + drop_node(toDropUniqueId, toDropIP, 9050, + 0, "", clusterName, cloudClusterId, ms) + // check have empty compute group + def msHttpPort = ms.host + ":" + ms.httpPort + def originalClusterId = "" + get_instance_api(msHttpPort, "default_instance_id") { + respCode, body -> + log.info("before drop node get instance resp: ${body} ${respCode}".toString()) + json = parseJson(body) + assertTrue(json.code.equalsIgnoreCase("OK")) + def clusters = json.result.clusters + assertTrue(clusters.any { cluster -> + cluster.cluster_name == clusterName && cluster.type == "COMPUTE" + }) + def ret = clusters.find { cluster -> + cluster.cluster_name == "compute_cluster1" && cluster.type == "COMPUTE" + } + originalClusterId = ret.cluster_id + assertNotEquals("", originalClusterId) + } + Thread.sleep(11000) + result = sql_return_maparray """SHOW COMPUTE GROUPS;""" + logger.info("show compute group : {}", result) + assertEquals(1, result.size()) + // after drop node, empty compute group not show + assertFalse(result.stream().anyMatch(cluster -> cluster.Name == """$clusterName""")) + + sql """ALTER SYSTEM RENAME COMPUTE GROUP compute_cluster1 $clusterName;""" + + result = sql_return_maparray """SHOW COMPUTE GROUPS;""" + logger.info("show compute group : {}", result) + assertEquals(1, result.size()) + assertTrue(result.stream().anyMatch(cluster -> cluster.Name == """$clusterName""")) + // check not have empty compute group + get_instance_api(msHttpPort, "default_instance_id") { + respCode, body -> + log.info("after drop node get instance resp: ${body} ${respCode}".toString()) + json = parseJson(body) + assertTrue(json.code.equalsIgnoreCase("OK")) + def clusters = json.result.clusters + assertTrue(clusters.any { cluster -> + cluster.cluster_name == clusterName && cluster.type == "COMPUTE" + }) + def ret = clusters.find { cluster -> + cluster.cluster_name == clusterName && cluster.type == "COMPUTE" + } + assertNotNull(ret) + // after rename compute group id not changed + assertEquals(originalClusterId, ret.cluster_id) + } + } +} --------------------------------------------------------------------- To unsubscribe, e-mail: commits-unsubscr...@doris.apache.org For additional commands, e-mail: commits-h...@doris.apache.org