Skip to content

Commit

Permalink
[feature] (recycle bin) Support delete catalog recycle bin (apache#31893
Browse files Browse the repository at this point in the history
)

Add a new command to support delete db/table/partition in catalog
recycle bin instantly.

For example:
1. Delete the database by id:
    DROP CATALOG RECYCLE BIN WHERE 'DbId' = db_id;

2. Delete the table by id:
    DROP CATALOG RECYCLE BIN WHERE 'TableId' = table_id;

3. Delete the partition by id:
    DROP CATALOG RECYCLE BIN WHERE 'PartitionId' = partition_id;
  • Loading branch information
Dragonliu2018 authored May 30, 2024
1 parent ffc80cf commit 5c747be
Show file tree
Hide file tree
Showing 9 changed files with 345 additions and 0 deletions.
Original file line number Diff line number Diff line change
Expand Up @@ -119,6 +119,7 @@ statementBase
| ALTER TABLE table=multipartIdentifier
DROP CONSTRAINT constraintName=errorCapturingIdentifier #dropConstraint
| SHOW CONSTRAINTS FROM table=multipartIdentifier #showConstraint
| DROP CATALOG RECYCLE BIN WHERE idType=STRING_LITERAL EQ id=INTEGER_VALUE #dropCatalogRecycleBin
| unsupportedStatement #unsupported
;

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -907,6 +907,117 @@ public synchronized void replayRecoverPartition(OlapTable table, long partitionI
}
}

// erase database in catalog recycle bin instantly
public synchronized void eraseDatabaseInstantly(long dbId) throws DdlException {
// 1. find dbInfo to erase
RecycleDatabaseInfo dbInfo = idToDatabase.get(dbId);
if (dbInfo == null) {
throw new DdlException("Unknown database id '" + dbId + "'");
}

// 2. erase db
Env.getCurrentEnv().eraseDatabase(dbId, true);

// 3. erase db from idToDatabase and idToRecycleTime
idToDatabase.remove(dbId);
idToRecycleTime.remove(dbId);

// 4. log for erase db
String dbName = dbInfo.getDb().getName();
LOG.info("erase db[{}]: {}", dbId, dbName);

// 5. remove all tables with the same dbId
List<Long> tableIdToErase = Lists.newArrayList();
Iterator<Map.Entry<Long, RecycleTableInfo>> tableIterator = idToTable.entrySet().iterator();
while (tableIterator.hasNext()) {
Map.Entry<Long, RecycleTableInfo> entry = tableIterator.next();
RecycleTableInfo tableInfo = entry.getValue();
if (tableInfo.getDbId() == dbId) {
tableIdToErase.add(entry.getKey());
}
}
for (Long tableId : tableIdToErase) {
eraseTableInstantly(tableId);
}

// 6. remove all partitions with the same dbId
List<Long> partitionIdToErase = Lists.newArrayList();
Iterator<Map.Entry<Long, RecyclePartitionInfo>> partitionIterator = idToPartition.entrySet().iterator();
while (partitionIterator.hasNext()) {
Map.Entry<Long, RecyclePartitionInfo> entry = partitionIterator.next();
RecyclePartitionInfo partitionInfo = entry.getValue();
if (partitionInfo.getDbId() == dbId) {
partitionIdToErase.add(entry.getKey());
}
}
for (Long partitionId : partitionIdToErase) {
erasePartitionInstantly(partitionId);
}
}

// erase table in catalog recycle bin instantly
public synchronized void eraseTableInstantly(long tableId) throws DdlException {
// 1. find tableInfo to erase
RecycleTableInfo tableInfo = idToTable.get(tableId);
if (tableInfo == null) {
throw new DdlException("Unknown table id '" + tableId + "'");
}

// 2. erase table
long dbId = tableInfo.getDbId();
Table table = tableInfo.getTable();
if (table.getType() == TableType.OLAP || table.getType() == TableType.MATERIALIZED_VIEW) {
Env.getCurrentEnv().onEraseOlapTable((OlapTable) table, false);
}

// 3. erase table from idToTable and idToRecycleTime
idToTable.remove(tableId);
idToRecycleTime.remove(tableId);

// 4. log for erase table
String tableName = table.getName();
Env.getCurrentEnv().getEditLog().logEraseTable(tableId);
LOG.info("erase db[{}]'s table[{}]: {}", dbId, tableId, tableName);


// 5. erase all partitions with the same tableId
List<Long> partitionIdToErase = Lists.newArrayList();
Iterator<Map.Entry<Long, RecyclePartitionInfo>> partitionIterator = idToPartition.entrySet().iterator();
while (partitionIterator.hasNext()) {
Map.Entry<Long, RecyclePartitionInfo> entry = partitionIterator.next();
RecyclePartitionInfo partitionInfo = entry.getValue();
if (partitionInfo.getTableId() == tableId) {
partitionIdToErase.add(entry.getKey());
}
}
for (Long partitionId : partitionIdToErase) {
erasePartitionInstantly(partitionId);
}
}

// erase partition in catalog recycle bin instantly
public synchronized void erasePartitionInstantly(long partitionId) throws DdlException {
// 1. find partitionInfo to erase
RecyclePartitionInfo partitionInfo = idToPartition.get(partitionId);
if (partitionInfo == null) {
throw new DdlException("No partition id '" + partitionId + "'");
}

// 2. erase partition
Partition partition = partitionInfo.getPartition();
Env.getCurrentEnv().onErasePartition(partition);

// 3. erase partition in idToPartition and idToRecycleTime
idToPartition.remove(partitionId);
idToRecycleTime.remove(partitionId);

// 4. log for erase partition
long tableId = partitionInfo.getTableId();
String partitionName = partition.getName();
Env.getCurrentEnv().getEditLog().logErasePartition(partitionId);
LOG.info("erase table[{}]'s partition[{}]: {}", tableId, partitionId, partitionName);
}

// no need to use synchronized.
// only called when loading image
public void addTabletToInvertedIndex() {
Expand Down
5 changes: 5 additions & 0 deletions fe/fe-core/src/main/java/org/apache/doris/catalog/Env.java
Original file line number Diff line number Diff line change
Expand Up @@ -191,6 +191,7 @@
import org.apache.doris.mysql.privilege.Auth;
import org.apache.doris.mysql.privilege.PrivPredicate;
import org.apache.doris.nereids.jobs.load.LabelProcessor;
import org.apache.doris.nereids.trees.plans.commands.DropCatalogRecycleBinCommand.IdType;
import org.apache.doris.nereids.trees.plans.commands.info.AlterMTMVPropertyInfo;
import org.apache.doris.nereids.trees.plans.commands.info.AlterMTMVRefreshInfo;
import org.apache.doris.nereids.trees.plans.commands.info.TableNameInfo;
Expand Down Expand Up @@ -3133,6 +3134,10 @@ public void recoverPartition(RecoverPartitionStmt recoverStmt) throws DdlExcepti
getInternalCatalog().recoverPartition(recoverStmt);
}

public void dropCatalogRecycleBin(IdType idType, long id) throws DdlException {
getInternalCatalog().dropCatalogRecycleBin(idType, id);
}

public void replayEraseDatabase(long dbId) throws DdlException {
Env.getCurrentRecycleBin().replayEraseDatabase(dbId);
}
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -143,6 +143,7 @@
import org.apache.doris.datasource.property.constants.HMSProperties;
import org.apache.doris.event.DropPartitionEvent;
import org.apache.doris.mysql.privilege.PrivPredicate;
import org.apache.doris.nereids.trees.plans.commands.DropCatalogRecycleBinCommand.IdType;
import org.apache.doris.nereids.trees.plans.commands.info.DropMTMVInfo;
import org.apache.doris.nereids.trees.plans.commands.info.TableNameInfo;
import org.apache.doris.persist.AlterDatabasePropertyInfo;
Expand Down Expand Up @@ -709,6 +710,26 @@ public void recoverPartition(RecoverPartitionStmt recoverStmt) throws DdlExcepti
}
}

public void dropCatalogRecycleBin(IdType idType, long id) throws DdlException {
switch (idType) {
case DATABASE_ID:
Env.getCurrentRecycleBin().eraseDatabaseInstantly(id);
LOG.info("drop database[{}] in catalog recycle bin", id);
break;
case TABLE_ID:
Env.getCurrentRecycleBin().eraseTableInstantly(id);
LOG.info("drop table[{}] in catalog recycle bin", id);
break;
case PARTITION_ID:
Env.getCurrentRecycleBin().erasePartitionInstantly(id);
LOG.info("drop partition[{}] in catalog recycle bin", id);
break;
default:
String message = "DROP CATALOG RECYCLE BIN: idType should be 'DbId', 'TableId' or 'PartitionId'.";
throw new DdlException(message);
}
}

public void replayRecoverDatabase(RecoverInfo info) {
long dbId = info.getDbId();
String newDbName = info.getNewDbName();
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -90,6 +90,7 @@
import org.apache.doris.nereids.DorisParser.DecimalLiteralContext;
import org.apache.doris.nereids.DorisParser.DeleteContext;
import org.apache.doris.nereids.DorisParser.DereferenceContext;
import org.apache.doris.nereids.DorisParser.DropCatalogRecycleBinContext;
import org.apache.doris.nereids.DorisParser.DropConstraintContext;
import org.apache.doris.nereids.DorisParser.DropMTMVContext;
import org.apache.doris.nereids.DorisParser.DropProcedureContext;
Expand Down Expand Up @@ -372,6 +373,8 @@
import org.apache.doris.nereids.trees.plans.commands.CreateViewCommand;
import org.apache.doris.nereids.trees.plans.commands.DeleteFromCommand;
import org.apache.doris.nereids.trees.plans.commands.DeleteFromUsingCommand;
import org.apache.doris.nereids.trees.plans.commands.DropCatalogRecycleBinCommand;
import org.apache.doris.nereids.trees.plans.commands.DropCatalogRecycleBinCommand.IdType;
import org.apache.doris.nereids.trees.plans.commands.DropConstraintCommand;
import org.apache.doris.nereids.trees.plans.commands.DropMTMVCommand;
import org.apache.doris.nereids.trees.plans.commands.DropProcedureCommand;
Expand Down Expand Up @@ -3552,6 +3555,15 @@ public LogicalPlan visitShowCreateProcedure(ShowCreateProcedureContext ctx) {
return ParserUtils.withOrigin(ctx, () -> new ShowCreateProcedureCommand(procedureName));
}

@Override
public LogicalPlan visitDropCatalogRecycleBin(DropCatalogRecycleBinContext ctx) {
String idTypeStr = ctx.idType.getText().substring(1, ctx.idType.getText().length() - 1);
IdType idType = IdType.fromString(idTypeStr);
long id = Long.parseLong(ctx.id.getText());

return ParserUtils.withOrigin(ctx, () -> new DropCatalogRecycleBinCommand(idType, id));
}

@Override
public Object visitUnsupported(UnsupportedContext ctx) {
return UnsupportedCommand.INSTANCE;
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -155,6 +155,7 @@ public enum PlanType {
SHOW_CREATE_PROCEDURE_COMMAND,
CREATE_VIEW_COMMAND,
ALTER_VIEW_COMMAND,
DROP_CATALOG_RECYCLE_BIN_COMMAND,
UNSUPPORTED_COMMAND,
CREATE_TABLE_LIKE_COMMAND
}
Original file line number Diff line number Diff line change
@@ -0,0 +1,81 @@
// 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.nereids.exceptions.AnalysisException;
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;

/**
* drop catalog recycle bin command
*/
public class DropCatalogRecycleBinCommand extends Command implements ForwardWithSync {

/**
* id type
*/
public enum IdType {
DATABASE_ID,
TABLE_ID,
PARTITION_ID;

/**
* return IdType according to given String
*/
public static IdType fromString(String idTypeStr) {
IdType idType;
if (idTypeStr.equalsIgnoreCase("DbId")) {
idType = DATABASE_ID;
} else if (idTypeStr.equalsIgnoreCase("TableId")) {
idType = TABLE_ID;
} else if (idTypeStr.equalsIgnoreCase("PartitionId")) {
idType = PARTITION_ID;
} else {
String message = "DROP CATALOG RECYCLE BIN: " + idTypeStr
+ " should be 'DbId', 'TableId' or 'PartitionId'.";
throw new AnalysisException(message);
}
return idType;
}
}

private final IdType idType;
private long id = -1;

/**
* constructor
*/
public DropCatalogRecycleBinCommand(IdType idType, long id) {
super(PlanType.DROP_CATALOG_RECYCLE_BIN_COMMAND);
this.idType = idType;
this.id = id;
}

@Override
public void run(ConnectContext ctx, StmtExecutor executor) throws Exception {
Env.getCurrentEnv().dropCatalogRecycleBin(idType, id);
}

@Override
public <R, C> R accept(PlanVisitor<R, C> visitor, C context) {
return visitor.visitDropCatalogRecycleBinCommand(this, context);
}
}
Original file line number Diff line number Diff line change
Expand Up @@ -31,6 +31,7 @@
import org.apache.doris.nereids.trees.plans.commands.CreateViewCommand;
import org.apache.doris.nereids.trees.plans.commands.DeleteFromCommand;
import org.apache.doris.nereids.trees.plans.commands.DeleteFromUsingCommand;
import org.apache.doris.nereids.trees.plans.commands.DropCatalogRecycleBinCommand;
import org.apache.doris.nereids.trees.plans.commands.DropConstraintCommand;
import org.apache.doris.nereids.trees.plans.commands.DropMTMVCommand;
import org.apache.doris.nereids.trees.plans.commands.DropProcedureCommand;
Expand Down Expand Up @@ -169,6 +170,10 @@ default R visitAlterViewCommand(AlterViewCommand alterViewCommand, C context) {
return visitCommand(alterViewCommand, context);
}

default R visitDropCatalogRecycleBinCommand(DropCatalogRecycleBinCommand dropCatalogRecycleBinCommand, C context) {
return visitCommand(dropCatalogRecycleBinCommand, context);
}

default R visitUnsupportedCommand(UnsupportedCommand unsupportedCommand, C context) {
return visitCommand(unsupportedCommand, context);
}
Expand Down
Loading

0 comments on commit 5c747be

Please sign in to comment.