You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@kudu.apache.org by al...@apache.org on 2022/08/13 00:56:50 UTC
[kudu] branch master updated: KUDU-3326 [delete]: Add soft delete table supports
This is an automated email from the ASF dual-hosted git repository.
alexey pushed a commit to branch master
in repository https://gitbox.apache.org/repos/asf/kudu.git
The following commit(s) were added to refs/heads/master by this push:
new 7b6b6b636 KUDU-3326 [delete]: Add soft delete table supports
7b6b6b636 is described below
commit 7b6b6b636818d3e22a3939fde77689dce84e88b2
Author: kedeng <kd...@gmail.com>
AuthorDate: Tue Oct 12 16:15:31 2021 +0800
KUDU-3326 [delete]: Add soft delete table supports
Kudu system will not delete the table immediately after receiving the
command to delete the table. Instead, it will mark the table and set
a validity period. After the validity period, will try again to
determine whether the table really needs to be deleted.
NOTE: Soft-delete related functions is not supported when HMS is enabled.
Change-Id: I3d1dddfbca55a5c4bcac4028157325ad618ea665
Reviewed-on: http://gerrit.cloudera.org:8080/17917
Tested-by: Kudu Jenkins
Reviewed-by: Alexey Serbin <al...@apache.org>
---
.../org/apache/kudu/client/AsyncKuduClient.java | 76 +++-
.../org/apache/kudu/client/DeleteTableRequest.java | 7 +-
.../java/org/apache/kudu/client/KuduClient.java | 73 +++-
.../org/apache/kudu/client/ListTablesRequest.java | 5 +
...Request.java => RecallDeletedTableRequest.java} | 49 ++-
.../kudu/client/RecallDeletedTableResponse.java | 33 ++
.../org/apache/kudu/client/TestKuduClient.java | 41 ++
src/kudu/client/client-internal.cc | 31 +-
src/kudu/client/client-internal.h | 13 +-
src/kudu/client/client-test.cc | 214 +++++++++-
src/kudu/client/client.cc | 37 +-
src/kudu/client/client.h | 66 +++-
src/kudu/client/master_proxy_rpc.cc | 3 +
src/kudu/client/master_proxy_rpc.h | 1 +
src/kudu/common/wire_protocol.cc | 4 -
src/kudu/common/wire_protocol.h | 4 +
src/kudu/integration-tests/master_hms-itest.cc | 27 ++
src/kudu/master/catalog_manager.cc | 440 +++++++++++++++++++--
src/kudu/master/catalog_manager.h | 107 ++++-
src/kudu/master/master-test.cc | 134 ++++++-
src/kudu/master/master.cc | 80 +++-
src/kudu/master/master.h | 9 +
src/kudu/master/master.proto | 33 ++
src/kudu/master/master_service.cc | 25 +-
src/kudu/master/master_service.h | 6 +
src/kudu/server/server_base.cc | 4 +-
src/kudu/server/server_base.h | 6 +-
src/kudu/tools/kudu-admin-test.cc | 57 ++-
src/kudu/tools/kudu-tool-test.cc | 61 ++-
src/kudu/tools/tool_action_table.cc | 37 +-
30 files changed, 1559 insertions(+), 124 deletions(-)
diff --git a/java/kudu-client/src/main/java/org/apache/kudu/client/AsyncKuduClient.java b/java/kudu-client/src/main/java/org/apache/kudu/client/AsyncKuduClient.java
index 303c641f9..d21ff80fd 100644
--- a/java/kudu-client/src/main/java/org/apache/kudu/client/AsyncKuduClient.java
+++ b/java/kudu-client/src/main/java/org/apache/kudu/client/AsyncKuduClient.java
@@ -302,6 +302,7 @@ public class AsyncKuduClient implements AutoCloseable {
public static final long NO_TIMESTAMP = -1;
public static final long INVALID_TXN_ID = -1;
public static final long DEFAULT_OPERATION_TIMEOUT_MS = 30000;
+ public static final int NO_SOFT_DELETED_STATE_RESERVED_SECONDS = 0;
public static final long DEFAULT_KEEP_ALIVE_PERIOD_MS = 15000; // 25% of the default scanner ttl.
public static final long DEFAULT_NEGOTIATION_TIMEOUT_MS = 10000;
private static final long MAX_RPC_ATTEMPTS = 100;
@@ -696,19 +697,58 @@ public class AsyncKuduClient implements AutoCloseable {
}
/**
- * Delete a table on the cluster with the specified name.
+ * Delete a table with the specified name. The table is purged immediately.
* @param name the table's name
* @return a deferred object to track the progress of the deleteTable command
*/
public Deferred<DeleteTableResponse> deleteTable(String name) {
+ return deleteTable(name, NO_SOFT_DELETED_STATE_RESERVED_SECONDS);
+ }
+
+ /**
+ * Delete a table with the specified name.
+ * @param name the table's name
+ * @param reserveSeconds the soft deleted table to be alive time
+ * @return a deferred object to track the progress of the deleteTable command
+ */
+ public Deferred<DeleteTableResponse> deleteTable(String name,
+ int reserveSeconds) {
checkIsClosed();
DeleteTableRequest delete = new DeleteTableRequest(this.masterTable,
name,
timer,
- defaultAdminOperationTimeoutMs);
+ defaultAdminOperationTimeoutMs,
+ reserveSeconds);
return sendRpcToTablet(delete);
}
+ /**
+ * Recall a soft-deleted table on the cluster with the specified id
+ * @param id the table's id
+ * @return a deferred object to track the progress of the recall command
+ */
+ public Deferred<RecallDeletedTableResponse> recallDeletedTable(String id) {
+ return recallDeletedTable(id, "");
+ }
+
+ /**
+ * Recall a soft-deleted table on the cluster with the specified id
+ * @param id the table's id
+ * @param newTableName the table's new name after recall
+ * @return a deferred object to track the progress of the recall command
+ */
+ public Deferred<RecallDeletedTableResponse> recallDeletedTable(String id,
+ String newTableName) {
+ checkIsClosed();
+ RecallDeletedTableRequest recall = new RecallDeletedTableRequest(
+ this.masterTable,
+ id,
+ newTableName,
+ timer,
+ defaultAdminOperationTimeoutMs);
+ return sendRpcToTablet(recall);
+ }
+
/**
* Alter a table on the cluster as specified by the builder.
*
@@ -856,27 +896,53 @@ public class AsyncKuduClient implements AutoCloseable {
}
/**
- * Get the list of all the tables.
+ * Get the list of all the regular (i.e. not soft-deleted) tables.
* @return a deferred object that yields a list of all the tables
*/
public Deferred<ListTablesResponse> getTablesList() {
- return getTablesList(null);
+ return getTablesList(null, false);
+ }
+
+ /**
+ * Get a list of regular table names. Passing a null filter returns all the tables. When a
+ * filter is specified, it only returns tables that satisfy a substring match.
+ * @param nameFilter an optional table name filter
+ * @return a deferred that yields the list of table names
+ */
+ public Deferred<ListTablesResponse> getTablesList(String nameFilter) {
+ ListTablesRequest rpc = new ListTablesRequest(this.masterTable,
+ nameFilter,
+ false,
+ timer,
+ defaultAdminOperationTimeoutMs);
+ return sendRpcToTablet(rpc);
}
/**
* Get a list of table names. Passing a null filter returns all the tables. When a filter is
* specified, it only returns tables that satisfy a substring match.
* @param nameFilter an optional table name filter
+ * @param showSoftDeleted whether to display only regular (i.e. not soft deleted)
+ * tables or all tables(i.e. soft deleted tables and regular tables)
* @return a deferred that yields the list of table names
*/
- public Deferred<ListTablesResponse> getTablesList(String nameFilter) {
+ public Deferred<ListTablesResponse> getTablesList(String nameFilter, boolean showSoftDeleted) {
ListTablesRequest rpc = new ListTablesRequest(this.masterTable,
nameFilter,
+ showSoftDeleted,
timer,
defaultAdminOperationTimeoutMs);
return sendRpcToTablet(rpc);
}
+ /**
+ * Get the list of all the soft deleted tables.
+ * @return a deferred object that yields a list of all the soft deleted tables
+ */
+ public Deferred<ListTablesResponse> getSoftDeletedTablesList() {
+ return getTablesList(null, true);
+ }
+
/**
* Get table's statistics from master.
* @param name the table's name
diff --git a/java/kudu-client/src/main/java/org/apache/kudu/client/DeleteTableRequest.java b/java/kudu-client/src/main/java/org/apache/kudu/client/DeleteTableRequest.java
index 127ec74cf..49b9ff461 100644
--- a/java/kudu-client/src/main/java/org/apache/kudu/client/DeleteTableRequest.java
+++ b/java/kudu-client/src/main/java/org/apache/kudu/client/DeleteTableRequest.java
@@ -34,12 +34,16 @@ class DeleteTableRequest extends KuduRpc<DeleteTableResponse> {
private final String name;
+ private final int reserveSeconds;
+
DeleteTableRequest(KuduTable table,
String name,
Timer timer,
- long timeoutMillis) {
+ long timeoutMillis,
+ int reserveSeconds) {
super(table, timer, timeoutMillis);
this.name = name;
+ this.reserveSeconds = reserveSeconds;
}
@Override
@@ -48,6 +52,7 @@ class DeleteTableRequest extends KuduRpc<DeleteTableResponse> {
Master.TableIdentifierPB tableID =
Master.TableIdentifierPB.newBuilder().setTableName(name).build();
builder.setTable(tableID);
+ builder.setReserveSeconds(reserveSeconds);
return builder.build();
}
diff --git a/java/kudu-client/src/main/java/org/apache/kudu/client/KuduClient.java b/java/kudu-client/src/main/java/org/apache/kudu/client/KuduClient.java
index 563181575..205167fa2 100644
--- a/java/kudu-client/src/main/java/org/apache/kudu/client/KuduClient.java
+++ b/java/kudu-client/src/main/java/org/apache/kudu/client/KuduClient.java
@@ -168,7 +168,47 @@ public class KuduClient implements AutoCloseable {
* @throws KuduException if anything went wrong
*/
public DeleteTableResponse deleteTable(String name) throws KuduException {
- Deferred<DeleteTableResponse> d = asyncClient.deleteTable(name);
+ Deferred<DeleteTableResponse> d = asyncClient.deleteTable(name, 0);
+ return joinAndHandleException(d);
+ }
+
+ /**
+ * SoftDelete a table on the cluster with the specified name, the table will be
+ * reserved for reserveSeconds before being purged.
+ * @param name the table's name
+ * @param reserveSeconds the soft deleted table to be alive time
+ * @return an rpc response object
+ * @throws KuduException if anything went wrong
+ */
+ public DeleteTableResponse deleteTable(String name,
+ int reserveSeconds) throws KuduException {
+ Deferred<DeleteTableResponse> d = asyncClient.deleteTable(name, reserveSeconds);
+ return joinAndHandleException(d);
+ }
+
+ /**
+ * Recall a deleted table on the cluster with the specified table id
+ * @param id the table's id
+ * @return an rpc response object
+ * @throws KuduException if anything went wrong
+ */
+ public RecallDeletedTableResponse recallDeletedTable(String id) throws KuduException {
+ Deferred<RecallDeletedTableResponse> d = asyncClient.recallDeletedTable(id);
+ return joinAndHandleException(d);
+ }
+
+ /**
+ * Recall a deleted table on the cluster with the specified table id
+ * and give the recalled table the new table name
+ * @param id the table's id
+ * @param newTableName the recalled table's new name
+ * @return an rpc response object
+ * @throws KuduException if anything went wrong
+ */
+ public RecallDeletedTableResponse recallDeletedTable(String id,
+ String newTableName) throws KuduException {
+ Deferred<RecallDeletedTableResponse> d = asyncClient.recallDeletedTable(id,
+ newTableName);
return joinAndHandleException(d);
}
@@ -219,8 +259,8 @@ public class KuduClient implements AutoCloseable {
}
/**
- * Get the list of all the tables.
- * @return a list of all the tables
+ * Get the list of all the regular tables.
+ * @return a list of all the regular tables
* @throws KuduException if anything went wrong
*/
public ListTablesResponse getTablesList() throws KuduException {
@@ -228,14 +268,35 @@ public class KuduClient implements AutoCloseable {
}
/**
- * Get a list of table names. Passing a null filter returns all the tables. When a filter is
- * specified, it only returns tables that satisfy a substring match.
+ * Get a list of regular table names. Passing a null filter returns all the tables.
+ * When a filter is specified, it only returns tables that satisfy a substring match.
* @param nameFilter an optional table name filter
* @return a deferred that contains the list of table names
* @throws KuduException if anything went wrong
*/
public ListTablesResponse getTablesList(String nameFilter) throws KuduException {
- Deferred<ListTablesResponse> d = asyncClient.getTablesList(nameFilter);
+ Deferred<ListTablesResponse> d = asyncClient.getTablesList(nameFilter, false);
+ return joinAndHandleException(d);
+ }
+
+ /**
+ * Get the list of all the soft deleted tables.
+ * @return a list of all the soft deleted tables
+ * @throws KuduException if anything went wrong
+ */
+ public ListTablesResponse getSoftDeletedTablesList() throws KuduException {
+ return getSoftDeletedTablesList(null);
+ }
+
+ /**
+ * Get list of soft deleted table names. Passing a null filter returns all the tables.
+ * When a filter is specified, it only returns tables that satisfy a substring match.
+ * @param nameFilter an optional table name filter
+ * @return a deferred that contains the list of table names
+ * @throws KuduException if anything went wrong
+ */
+ public ListTablesResponse getSoftDeletedTablesList(String nameFilter) throws KuduException {
+ Deferred<ListTablesResponse> d = asyncClient.getTablesList(nameFilter, true);
return joinAndHandleException(d);
}
diff --git a/java/kudu-client/src/main/java/org/apache/kudu/client/ListTablesRequest.java b/java/kudu-client/src/main/java/org/apache/kudu/client/ListTablesRequest.java
index a462ba177..698a4e521 100644
--- a/java/kudu-client/src/main/java/org/apache/kudu/client/ListTablesRequest.java
+++ b/java/kudu-client/src/main/java/org/apache/kudu/client/ListTablesRequest.java
@@ -33,12 +33,16 @@ class ListTablesRequest extends KuduRpc<ListTablesResponse> {
private final String nameFilter;
+ private final boolean showSoftDeleted;
+
ListTablesRequest(KuduTable masterTable,
String nameFilter,
+ boolean showSoftDeleted,
Timer timer,
long timeoutMillis) {
super(masterTable, timer, timeoutMillis);
this.nameFilter = nameFilter;
+ this.showSoftDeleted = showSoftDeleted;
}
@Override
@@ -48,6 +52,7 @@ class ListTablesRequest extends KuduRpc<ListTablesResponse> {
if (nameFilter != null) {
builder.setNameFilter(nameFilter);
}
+ builder.setShowSoftDeleted(showSoftDeleted);
return builder.build();
}
diff --git a/java/kudu-client/src/main/java/org/apache/kudu/client/DeleteTableRequest.java b/java/kudu-client/src/main/java/org/apache/kudu/client/RecallDeletedTableRequest.java
similarity index 50%
copy from java/kudu-client/src/main/java/org/apache/kudu/client/DeleteTableRequest.java
copy to java/kudu-client/src/main/java/org/apache/kudu/client/RecallDeletedTableRequest.java
index 127ec74cf..61f00f403 100644
--- a/java/kudu-client/src/main/java/org/apache/kudu/client/DeleteTableRequest.java
+++ b/java/kudu-client/src/main/java/org/apache/kudu/client/RecallDeletedTableRequest.java
@@ -17,6 +17,7 @@
package org.apache.kudu.client;
+import com.google.protobuf.ByteString;
import com.google.protobuf.Message;
import io.netty.util.Timer;
import org.apache.yetus.audience.InterfaceAudience;
@@ -25,29 +26,36 @@ import org.apache.kudu.master.Master;
import org.apache.kudu.util.Pair;
/**
- * RPC to delete tables
+ * RPC to recall tables
*/
@InterfaceAudience.Private
-class DeleteTableRequest extends KuduRpc<DeleteTableResponse> {
+class RecallDeletedTableRequest extends KuduRpc<RecallDeletedTableResponse> {
- static final String DELETE_TABLE = "DeleteTable";
+ static final String RECALL_DELETED_TABLE = "RecallDeletedTable";
- private final String name;
+ private final String newTableName;
- DeleteTableRequest(KuduTable table,
- String name,
- Timer timer,
- long timeoutMillis) {
+ private final String id;
+
+ RecallDeletedTableRequest(KuduTable table,
+ String id,
+ String newTableName,
+ Timer timer,
+ long timeoutMillis) {
super(table, timer, timeoutMillis);
- this.name = name;
+ this.id = id;
+ this.newTableName = newTableName;
}
@Override
Message createRequestPB() {
- final Master.DeleteTableRequestPB.Builder builder = Master.DeleteTableRequestPB.newBuilder();
- Master.TableIdentifierPB tableID =
- Master.TableIdentifierPB.newBuilder().setTableName(name).build();
- builder.setTable(tableID);
+ final Master.RecallDeletedTableRequestPB.Builder builder =
+ Master.RecallDeletedTableRequestPB.newBuilder();
+ builder.setTable(Master.TableIdentifierPB.newBuilder()
+ .setTableId(ByteString.copyFromUtf8(id)));
+ if (!newTableName.isEmpty()) {
+ builder.setNewTableName(newTableName);
+ }
return builder.build();
}
@@ -58,17 +66,18 @@ class DeleteTableRequest extends KuduRpc<DeleteTableResponse> {
@Override
String method() {
- return DELETE_TABLE;
+ return RECALL_DELETED_TABLE;
}
@Override
- Pair<DeleteTableResponse, Object> deserialize(CallResponse callResponse,
- String tsUUID) throws KuduException {
- final Master.DeleteTableResponsePB.Builder builder = Master.DeleteTableResponsePB.newBuilder();
+ Pair<RecallDeletedTableResponse, Object> deserialize(CallResponse callResponse,
+ String tsUUID) throws KuduException {
+ final Master.RecallDeletedTableResponsePB.Builder builder =
+ Master.RecallDeletedTableResponsePB.newBuilder();
readProtobuf(callResponse.getPBMessage(), builder);
- DeleteTableResponse response =
- new DeleteTableResponse(timeoutTracker.getElapsedMillis(), tsUUID);
- return new Pair<DeleteTableResponse, Object>(
+ RecallDeletedTableResponse response =
+ new RecallDeletedTableResponse(timeoutTracker.getElapsedMillis(), tsUUID);
+ return new Pair<RecallDeletedTableResponse, Object>(
response, builder.hasError() ? builder.getError() : null);
}
}
diff --git a/java/kudu-client/src/main/java/org/apache/kudu/client/RecallDeletedTableResponse.java b/java/kudu-client/src/main/java/org/apache/kudu/client/RecallDeletedTableResponse.java
new file mode 100644
index 000000000..c30fa051e
--- /dev/null
+++ b/java/kudu-client/src/main/java/org/apache/kudu/client/RecallDeletedTableResponse.java
@@ -0,0 +1,33 @@
+// 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.kudu.client;
+
+import org.apache.yetus.audience.InterfaceAudience;
+import org.apache.yetus.audience.InterfaceStability;
+
+@InterfaceAudience.Public
+@InterfaceStability.Evolving
+public class RecallDeletedTableResponse extends KuduRpcResponse {
+
+ /**
+ * @param elapsedMillis Time in milliseconds since RPC creation to now.
+ */
+ RecallDeletedTableResponse(long elapsedMillis, String tsUUID) {
+ super(elapsedMillis, tsUUID);
+ }
+}
diff --git a/java/kudu-client/src/test/java/org/apache/kudu/client/TestKuduClient.java b/java/kudu-client/src/test/java/org/apache/kudu/client/TestKuduClient.java
index 82670b812..01965daff 100644
--- a/java/kudu-client/src/test/java/org/apache/kudu/client/TestKuduClient.java
+++ b/java/kudu-client/src/test/java/org/apache/kudu/client/TestKuduClient.java
@@ -172,6 +172,47 @@ public class TestKuduClient {
newSchema.getColumn("column3_s").getCompressionAlgorithm());
}
+ /**
+ * Test recalling a soft deleted table through a KuduClient.
+ */
+ @Test(timeout = 100000)
+ public void testRecallDeletedTable() throws Exception {
+ // Check that we can create a table.
+ assertTrue(client.getTablesList().getTablesList().isEmpty());
+ final KuduTable table = client.createTable(TABLE_NAME, basicSchema,
+ getBasicCreateTableOptions());
+ final String tableId = table.getTableId();
+ assertEquals(1, client.getTablesList().getTablesList().size());
+ assertEquals(TABLE_NAME, client.getTablesList().getTablesList().get(0));
+
+ // Check that we can delete it.
+ client.deleteTable(TABLE_NAME, 600);
+ List<String> tables = client.getTablesList().getTablesList();
+ assertEquals(0, tables.size());
+ tables = client.getSoftDeletedTablesList().getTablesList();
+ assertEquals(1, tables.size());
+ String softDeletedTable = tables.get(0);
+ assertEquals(TABLE_NAME, softDeletedTable);
+ // Check that we can recall the soft_deleted table.
+ client.recallDeletedTable(tableId);
+ assertEquals(1, client.getTablesList().getTablesList().size());
+ assertEquals(TABLE_NAME, client.getTablesList().getTablesList().get(0));
+
+ // Check that we can delete it.
+ client.deleteTable(TABLE_NAME, 600);
+ tables = client.getTablesList().getTablesList();
+ assertEquals(0, tables.size());
+ tables = client.getSoftDeletedTablesList().getTablesList();
+ assertEquals(1, tables.size());
+ softDeletedTable = tables.get(0);
+ assertEquals(TABLE_NAME, softDeletedTable);
+ // Check we can recall soft deleted table with new table name.
+ final String newTableName = "NewTable";
+ client.recallDeletedTable(tableId, newTableName);
+ assertEquals(1, client.getTablesList().getTablesList().size());
+ assertEquals(newTableName, client.getTablesList().getTablesList().get(0));
+ }
+
/**
* Test creating a table with various invalid schema cases.
*/
diff --git a/src/kudu/client/client-internal.cc b/src/kudu/client/client-internal.cc
index 0c8a6f7bb..321576ad2 100644
--- a/src/kudu/client/client-internal.cc
+++ b/src/kudu/client/client-internal.cc
@@ -98,6 +98,8 @@ using kudu::master::ListTabletServersRequestPB;
using kudu::master::ListTabletServersResponsePB;
using kudu::master::MasterFeatures;
using kudu::master::MasterServiceProxy;
+using kudu::master::RecallDeletedTableRequestPB;
+using kudu::master::RecallDeletedTableResponsePB;
using kudu::master::TableIdentifierPB;
using kudu::rpc::BackoffType;
using kudu::rpc::CredentialsPolicy;
@@ -373,12 +375,14 @@ Status KuduClient::Data::WaitForCreateTableToFinish(
Status KuduClient::Data::DeleteTable(KuduClient* client,
const string& table_name,
const MonoTime& deadline,
- bool modify_external_catalogs) {
+ bool modify_external_catalogs,
+ uint32_t reserve_seconds) {
DeleteTableRequestPB req;
DeleteTableResponsePB resp;
req.mutable_table()->set_table_name(table_name);
req.set_modify_external_catalogs(modify_external_catalogs);
+ req.set_reserve_seconds(reserve_seconds);
Synchronizer sync;
AsyncLeaderMasterRpc<DeleteTableRequestPB, DeleteTableResponsePB> rpc(
deadline, client, BackoffType::EXPONENTIAL, req, &resp,
@@ -387,6 +391,26 @@ Status KuduClient::Data::DeleteTable(KuduClient* client,
return sync.Wait();
}
+Status KuduClient::Data::RecallTable(KuduClient* client,
+ const std::string& table_id,
+ const MonoTime& deadline,
+ const std::string& new_table_name) {
+ RecallDeletedTableRequestPB req;
+ RecallDeletedTableResponsePB resp;
+
+ req.mutable_table()->set_table_id(table_id);
+ if (!new_table_name.empty()) {
+ req.set_new_table_name(new_table_name);
+ }
+ Synchronizer sync;
+ AsyncLeaderMasterRpc<RecallDeletedTableRequestPB, RecallDeletedTableResponsePB> rpc(
+ deadline, client, BackoffType::EXPONENTIAL, req, &resp,
+ &MasterServiceProxy::RecallDeletedTableAsync, "RecallDeletedTable", sync.AsStatusCallback(),
+ {});
+ rpc.SendRpc();
+ return sync.Wait();
+}
+
Status KuduClient::Data::AlterTable(KuduClient* client,
const AlterTableRequestPB& req,
AlterTableResponsePB* resp,
@@ -445,12 +469,13 @@ Status KuduClient::Data::WaitForAlterTableToFinish(
Status KuduClient::Data::ListTablesWithInfo(KuduClient* client,
vector<TableInfo>* tables_info,
const string& filter,
- bool list_tablet_with_partition) {
+ bool list_tablet_with_partition,
+ bool show_soft_deleted) {
ListTablesRequestPB req;
if (!filter.empty()) {
req.set_name_filter(filter);
}
-
+ req.set_show_soft_deleted(show_soft_deleted);
req.set_list_tablet_with_partition(list_tablet_with_partition);
auto deadline = MonoTime::Now() + client->default_admin_operation_timeout();
diff --git a/src/kudu/client/client-internal.h b/src/kudu/client/client-internal.h
index 8a062b2c8..3e820f91f 100644
--- a/src/kudu/client/client-internal.h
+++ b/src/kudu/client/client-internal.h
@@ -83,6 +83,8 @@ class RemoteTabletServer;
class KuduClient::Data {
public:
+ static const uint32_t kSoftDeletedTableReservationSeconds = 60 * 60 * 24 * 7;
+
Data();
~Data();
@@ -120,7 +122,13 @@ class KuduClient::Data {
static Status DeleteTable(KuduClient* client,
const std::string& table_name,
const MonoTime& deadline,
- bool modify_external_catalogs = true);
+ bool modify_external_catalogs = true,
+ uint32_t reserve_seconds = kSoftDeletedTableReservationSeconds);
+
+ static Status RecallTable(KuduClient* client,
+ const std::string& table_id,
+ const MonoTime& deadline,
+ const std::string& new_table_name = "");
static Status AlterTable(KuduClient* client,
const master::AlterTableRequestPB& req,
@@ -154,7 +162,8 @@ class KuduClient::Data {
static Status ListTablesWithInfo(KuduClient* client,
std::vector<TableInfo>* tables_info,
const std::string& filter,
- bool list_tablet_with_partition = false);
+ bool list_tablet_with_partition = false,
+ bool show_soft_deleted = false);
// Open the table identified by 'table_identifier'.
Status OpenTable(KuduClient* client,
diff --git a/src/kudu/client/client-test.cc b/src/kudu/client/client-test.cc
index 9946afd1f..f649cd488 100644
--- a/src/kudu/client/client-test.cc
+++ b/src/kudu/client/client-test.cc
@@ -74,6 +74,7 @@
#include "kudu/common/schema.h"
#include "kudu/common/timestamp.h"
#include "kudu/common/txn_id.h"
+#include "kudu/common/wire_protocol.h"
#include "kudu/common/wire_protocol.pb.h"
#include "kudu/consensus/metadata.pb.h"
#include "kudu/gutil/atomicops.h"
@@ -152,6 +153,7 @@ DECLARE_bool(scanner_inject_service_unavailable_on_continue_scan);
DECLARE_bool(txn_manager_enabled);
DECLARE_bool(txn_manager_lazily_initialized);
DECLARE_int32(client_tablet_locations_by_id_ttl_ms);
+DECLARE_int32(check_expired_table_interval_seconds);
DECLARE_int32(flush_threshold_mb);
DECLARE_int32(flush_threshold_secs);
DECLARE_int32(heartbeat_interval_ms);
@@ -249,6 +251,7 @@ class ClientTest : public KuduTest {
// Reduce the TS<->Master heartbeat interval
FLAGS_heartbeat_interval_ms = 10;
FLAGS_scanner_gc_check_interval_us = 50 * 1000; // 50 milliseconds.
+ FLAGS_check_expired_table_interval_seconds = 1;
// Enable TxnManager in Kudu master.
FLAGS_txn_manager_enabled = true;
@@ -893,10 +896,16 @@ TEST_F(ClientTest, TestListTables) {
std::sort(tables.begin(), tables.end());
ASSERT_EQ(string(kTableName), tables[0]);
ASSERT_EQ(string(kTable2Name), tables[1]);
- tables.clear();
ASSERT_OK(client_->ListTables(&tables, "testtb2"));
ASSERT_EQ(1, tables.size());
ASSERT_EQ(string(kTable2Name), tables[0]);
+
+ ASSERT_OK(client_->ListSoftDeletedTables(&tables));
+ ASSERT_EQ(0, tables.size());
+
+ ASSERT_OK(client_->SoftDeleteTable(kTable2Name, 1000));
+ ASSERT_OK(client_->ListSoftDeletedTables(&tables));
+ ASSERT_EQ(1, tables.size());
}
TEST_F(ClientTest, TestListTabletServers) {
@@ -4950,6 +4959,209 @@ TEST_F(ClientTest, TestDeleteTable) {
NO_FATALS(InsertTestRows(client_.get(), client_table_.get(), 10));
}
+TEST_F(ClientTest, TestSoftDeleteAndReserveTable) {
+ // Open the table before deleting it.
+ ASSERT_OK(client_->OpenTable(kTableName, &client_table_));
+ string table_id = client_table_->id();
+
+ // Insert a few rows, and scan them back. This is to populate the MetaCache.
+ NO_FATALS(InsertTestRows(client_.get(), client_table_.get(), 10));
+ vector<string> rows;
+ ScanTableToStrings(client_table_.get(), &rows);
+ ASSERT_EQ(10, rows.size());
+
+ // Remove the table.
+ // NOTE that it returns when the operation is completed on the master side
+ string tablet_id = GetFirstTabletId(client_table_.get());
+ ASSERT_OK(client_->SoftDeleteTable(kTableName, 600));
+ CatalogManager* catalog_manager = cluster_->mini_master()->master()->catalog_manager();
+ {
+ CatalogManager::ScopedLeaderSharedLock l(catalog_manager);
+ ASSERT_OK(l.first_failed_status());
+ bool exists;
+ ASSERT_OK(catalog_manager->TableNameExists(kTableName, &exists));
+ ASSERT_TRUE(exists);
+ }
+
+ // Soft_deleted tablet is still visible.
+ scoped_refptr<TabletReplica> tablet_replica;
+ ASSERT_TRUE(cluster_->mini_tablet_server(0)->server()->tablet_manager()->LookupTablet(
+ tablet_id, &tablet_replica));
+
+ // Old table has been removed to the soft_deleted map.
+ vector<string> tables;
+ ASSERT_OK(client_->ListTables(&tables));
+ ASSERT_EQ(0, tables.size());
+ ASSERT_OK(client_->ListSoftDeletedTables(&tables));
+ ASSERT_EQ(1, tables.size());
+ ASSERT_EQ(string(kTableName), tables[0]);
+
+ Status s;
+ // Altering a soft-deleted table is not allowed.
+ {
+ // Not allowed to rename.
+ unique_ptr<KuduTableAlterer> table_alterer(client_->NewTableAlterer(kTableName));
+ const string new_name = "test-new-table";
+ table_alterer->RenameTo(kTableName);
+ s = table_alterer->Alter();
+ ASSERT_TRUE(s.IsInvalidArgument());
+ ASSERT_STR_CONTAINS(s.ToString(), Substitute("soft_deleted table $0 should not be altered",
+ kTableName));
+ }
+
+ {
+ // Not allowed to add column.
+ unique_ptr<KuduTableAlterer> table_alterer(client_->NewTableAlterer(kTableName));
+ table_alterer->DropColumn("int_val")
+ ->AddColumn("new_col")->Type(KuduColumnSchema::INT32);
+ s = table_alterer->Alter();
+ ASSERT_TRUE(s.IsInvalidArgument());
+ ASSERT_STR_CONTAINS(s.ToString(), Substitute("soft_deleted table $0 should not be altered",
+ kTableName));
+ }
+
+ {
+ // Not allowed to delete the soft_deleted table with new reserve_seconds value.
+ s = client_->SoftDeleteTable(kTableName, 600);
+ ASSERT_TRUE(s.IsInvalidArgument());
+ ASSERT_STR_CONTAINS(s.ToString(), Substitute("soft_deleted table $0 should not be deleted",
+ kTableName));
+ }
+
+ {
+ // Not allowed to set extra configs.
+ map<string, string> extra_configs;
+ extra_configs[kTableMaintenancePriority] = "3";
+ unique_ptr<KuduTableAlterer> table_alterer(client_->NewTableAlterer(kTableName));
+ table_alterer->AlterExtraConfig(extra_configs);
+ s = table_alterer->Alter();
+ ASSERT_TRUE(s.IsInvalidArgument());
+ ASSERT_STR_CONTAINS(s.ToString(), Substitute("soft_deleted table $0 should not be altered",
+ kTableName));
+ }
+
+ {
+ // Read and write are allowed for soft_deleted table.
+ NO_FATALS(InsertTestRows(client_.get(), client_table_.get(), 20, 10));
+ ScanTableToStrings(client_table_.get(), &rows);
+ ASSERT_EQ(30, rows.size());
+ }
+
+ {
+ // Not allowed creating a new table with the same name.
+ shared_ptr<KuduTable> new_client_table;
+ s = CreateTable(kTableName, 1, GenerateSplitRows(), {}, &new_client_table);
+ ASSERT_TRUE(s.IsAlreadyPresent());
+ ASSERT_STR_CONTAINS(s.ToString(), Substitute("table $0 already exists with id",
+ kTableName));
+ }
+
+ // Try to recall the soft_deleted table.
+ ASSERT_OK(client_->RecallTable(table_id));
+ ASSERT_OK(client_->ListTables(&tables));
+ ASSERT_EQ(1, tables.size());
+ ASSERT_EQ(kTableName, tables[0]);
+ ASSERT_OK(client_->ListSoftDeletedTables(&tables));
+ ASSERT_TRUE(tables.empty());
+
+ // Force to delete the soft_deleted table.
+ ASSERT_OK(client_->SoftDeleteTable(kTableName));
+
+ // No one table left.
+ ASSERT_OK(client_->ListTables(&tables));
+ ASSERT_EQ(0, tables.size());
+ ASSERT_OK(client_->ListSoftDeletedTables(&tables));
+ ASSERT_EQ(0, tables.size());
+}
+
+TEST_F(ClientTest, TestSoftDeleteAndRecallTable) {
+ // Open the table before deleting it.
+ ASSERT_OK(client_->OpenTable(kTableName, &client_table_));
+
+ // Insert a few rows, and scan them back. This is to populate the MetaCache.
+ NO_FATALS(InsertTestRows(client_.get(), client_table_.get(), 10));
+ vector<string> rows;
+ ScanTableToStrings(client_table_.get(), &rows);
+ ASSERT_EQ(10, rows.size());
+
+ // Remove the table
+ ASSERT_OK(client_->SoftDeleteTable(kTableName, 600));
+ vector<string> tables;
+ ASSERT_OK(client_->ListTables(&tables));
+ ASSERT_EQ(0, tables.size());
+
+ // Recall and reopen table.
+ ASSERT_OK(client_->ListSoftDeletedTables(&tables));
+ ASSERT_EQ(1, tables.size());
+ ASSERT_OK(client_->RecallTable(client_table_->id()));
+
+ // Check the data in the table.
+ ScanTableToStrings(client_table_.get(), &rows);
+ ASSERT_EQ(10, rows.size());
+}
+
+TEST_F(ClientTest, TestSoftDeleteAndRecallTableWithNewTableName) {
+ // Open the table before deleting it.
+ ASSERT_OK(client_->OpenTable(kTableName, &client_table_));
+ string old_table_id = client_table_->id();
+
+ // Insert a few rows, and scan them back. This is to populate the MetaCache.
+ NO_FATALS(InsertTestRows(client_.get(), client_table_.get(), 10));
+ vector<string> rows;
+ ScanTableToStrings(client_table_.get(), &rows);
+ ASSERT_EQ(10, rows.size());
+
+ // Remove the table
+ ASSERT_OK(client_->SoftDeleteTable(kTableName, 600));
+ vector<string> tables;
+ ASSERT_OK(client_->ListTables(&tables));
+ ASSERT_EQ(0, tables.size());
+
+ // Recall and reopen table.
+ ASSERT_OK(client_->ListSoftDeletedTables(&tables));
+ ASSERT_EQ(1, tables.size());
+
+ const string new_name = "test-new-table";
+ ASSERT_OK(client_->RecallTable(client_table_->id(), new_name));
+ ASSERT_OK(client_->OpenTable(new_name, &client_table_));
+ ASSERT_EQ(old_table_id, client_table_->id());
+
+ // Check the data in the table.
+ ScanTableToStrings(client_table_.get(), &rows);
+ ASSERT_EQ(10, rows.size());
+}
+
+TEST_F(ClientTest, TestSoftDeleteAndRecallAfterReserveTimeTable) {
+ // Open the table before deleting it.
+ ASSERT_OK(client_->OpenTable(kTableName, &client_table_));
+
+ // Insert a few rows, and scan them back. This is to populate the MetaCache.
+ NO_FATALS(InsertTestRows(client_.get(), client_table_.get(), 10));
+ vector<string> rows;
+ ScanTableToStrings(client_table_.get(), &rows);
+ ASSERT_EQ(10, rows.size());
+
+ // Remove the table
+ ASSERT_OK(client_->SoftDeleteTable(kTableName));
+ // Wait util the table is removed completely.
+ ASSERT_EVENTUALLY([&] () {
+ Status s = client_->OpenTable(kTableName, &client_table_);
+ ASSERT_TRUE(s.IsNotFound());
+ ASSERT_STR_CONTAINS(s.ToString(), "the table does not exist");
+ });
+
+ vector<string> tables;
+ ASSERT_OK(client_->ListSoftDeletedTables(&tables));
+ ASSERT_EQ(0, tables.size());
+
+ // Try to recall the table.
+ Status s = client_->RecallTable(client_table_->id());
+ ASSERT_TRUE(s.IsNotFound());
+
+ ASSERT_OK(client_->ListTables(&tables));
+ ASSERT_TRUE(tables.empty());
+}
+
TEST_F(ClientTest, TestGetTableSchema) {
KuduSchema schema;
diff --git a/src/kudu/client/client.cc b/src/kudu/client/client.cc
index d31214371..47cb589cd 100644
--- a/src/kudu/client/client.cc
+++ b/src/kudu/client/client.cc
@@ -33,7 +33,6 @@
#include <glog/logging.h>
#include <google/protobuf/stubs/common.h>
-#include <google/protobuf/stubs/port.h>
#include "kudu/client/callbacks.h"
#include "kudu/client/client-internal.h"
@@ -83,7 +82,6 @@
#include "kudu/master/master.proxy.h"
#include "kudu/rpc/messenger.h"
#include "kudu/rpc/request_tracker.h"
-#include "kudu/rpc/response_callback.h"
#include "kudu/rpc/rpc.h"
#include "kudu/rpc/rpc_controller.h"
#include "kudu/rpc/sasl_common.h"
@@ -92,7 +90,7 @@
#include "kudu/security/tls_context.h"
#include "kudu/security/token.pb.h"
#include "kudu/tserver/tserver.pb.h"
-#include "kudu/tserver/tserver_service.proxy.h"
+#include "kudu/tserver/tserver_service.proxy.h" // IWYU pragma: keep
#include "kudu/util/async_util.h"
#include "kudu/util/debug-util.h"
#include "kudu/util/init.h"
@@ -511,13 +509,25 @@ Status KuduClient::IsCreateTableInProgress(const string& table_name,
}
Status KuduClient::DeleteTable(const string& table_name) {
- return DeleteTableInCatalogs(table_name, true);
+ return SoftDeleteTable(table_name);
+}
+
+Status KuduClient::SoftDeleteTable(const string& table_name,
+ uint32_t reserve_seconds) {
+ return DeleteTableInCatalogs(table_name, true, reserve_seconds);
}
Status KuduClient::DeleteTableInCatalogs(const string& table_name,
- bool modify_external_catalogs) {
+ bool modify_external_catalogs,
+ uint32_t reserve_seconds) {
MonoTime deadline = MonoTime::Now() + default_admin_operation_timeout();
- return data_->DeleteTable(this, table_name, deadline, modify_external_catalogs);
+ return KuduClient::Data::DeleteTable(this, table_name, deadline, modify_external_catalogs,
+ reserve_seconds);
+}
+
+Status KuduClient::RecallTable(const string& table_id, const string& new_table_name) {
+ MonoTime deadline = MonoTime::Now() + default_admin_operation_timeout();
+ return KuduClient::Data::RecallTable(this, table_id, deadline, new_table_name);
}
KuduTableAlterer* KuduClient::NewTableAlterer(const string& table_name) {
@@ -567,9 +577,22 @@ Status KuduClient::ListTabletServers(vector<KuduTabletServer*>* tablet_servers)
}
Status KuduClient::ListTables(vector<string>* tables, const string& filter) {
+ vector<Data::TableInfo> tables_info;
+ RETURN_NOT_OK(data_->ListTablesWithInfo(this, &tables_info, filter, false));
tables->clear();
+ tables->reserve(tables_info.size());
+ for (auto& info : tables_info) {
+ tables->emplace_back(std::move(info.table_name));
+ }
+ return Status::OK();
+}
+
+Status KuduClient::ListSoftDeletedTables(vector<string>* tables, const string& filter) {
vector<Data::TableInfo> tables_info;
- RETURN_NOT_OK(data_->ListTablesWithInfo(this, &tables_info, filter));
+ RETURN_NOT_OK(data_->ListTablesWithInfo(this, &tables_info, filter,
+ /*list_tablet_with_partition=*/ true, /*show_soft_deleted=*/ true));
+ tables->clear();
+ tables->reserve(tables_info.size());
for (auto& info : tables_info) {
tables->emplace_back(std::move(info.table_name));
}
diff --git a/src/kudu/client/client.h b/src/kudu/client/client.h
index 26cdf9cb8..f39cd8f3c 100644
--- a/src/kudu/client/client.h
+++ b/src/kudu/client/client.h
@@ -681,13 +681,47 @@ class KUDU_EXPORT KuduClient : public sp::enable_shared_from_this<KuduClient> {
Status IsCreateTableInProgress(const std::string& table_name,
bool* create_in_progress);
- /// Delete/drop a table.
+ /// Delete/drop a table without reserving.
+ ///
+ /// The delete operation or drop operation means that the service will directly
+ /// delete the table after receiving the instruction. Which means that once we
+ /// delete the table by mistake, we have no way to recall the deleted data.
+ /// We have added a new API @SoftDeleteTable to allow the deleted data to be
+ /// reserved for a period of time, which means that the wrongly deleted data may
+ /// be recalled. In order to be compatible with the previous versions, this interface
+ /// will continue to directly delete tables without reserving the table.
+ ///
+ /// Refer to SoftDeleteTable for detailed usage examples.
///
/// @param [in] table_name
/// Name of the table to drop.
/// @return Operation status.
Status DeleteTable(const std::string& table_name);
+ /// Soft delete/drop a table.
+ ///
+ /// Usage Example1:
+ /// Equal to DeleteTable(table_name) and the table will not be reserved.
+ /// @code
+ /// client->SoftDeleteTable(table_name);
+ /// @endcode
+ ///
+ /// Usage Example2:
+ /// The table will be reserved for 600s after delete operation.
+ /// We can recall the table in time after the delete.
+ /// @code
+ /// client->SoftDeleteTable(table_name, false, 600);
+ /// client->RecallTable(table_id);
+ /// @endcode
+
+ /// @param [in] table_name
+ /// Name of the table to drop.
+ /// @param [in] reserve_seconds
+ /// Reserve seconds after being deleted.
+ /// @return Operation status.
+ Status SoftDeleteTable(const std::string& table_name,
+ uint32_t reserve_seconds = 0);
+
/// @cond PRIVATE_API
/// Delete/drop a table in internal catalogs and possibly external catalogs.
@@ -699,9 +733,23 @@ class KUDU_EXPORT KuduClient : public sp::enable_shared_from_this<KuduClient> {
/// @param [in] modify_external_catalogs
/// Whether to apply the deletion to external catalogs, such as the Hive Metastore,
/// which the Kudu master has been configured to integrate with.
+ /// @param [in] reserve_seconds
+ /// Reserve seconds after being deleted.
/// @return Operation status.
Status DeleteTableInCatalogs(const std::string& table_name,
- bool modify_external_catalogs) KUDU_NO_EXPORT;
+ bool modify_external_catalogs,
+ uint32_t reserve_seconds = 0) KUDU_NO_EXPORT;
+
+ /// Recall a deleted but still reserved table.
+ ///
+ /// @param [in] table_id
+ /// ID of the table to recall.
+ /// @param [in] new_table_name
+ /// New table name for the recalled table. The recalled table will use the original
+ /// table name if the parameter is empty string (i.e. "").
+ /// @return Operation status.
+ Status RecallTable(const std::string& table_id, const std::string& new_table_name = "");
+
/// @endcond
/// Create a KuduTableAlterer object.
@@ -740,7 +788,8 @@ class KUDU_EXPORT KuduClient : public sp::enable_shared_from_this<KuduClient> {
/// @return Operation status.
Status ListTabletServers(std::vector<KuduTabletServer*>* tablet_servers);
- /// List only those tables whose names pass a substring match on @c filter.
+ /// List non-soft-deleted tables whose names pass a substring
+ /// match on @c filter.
///
/// @param [out] tables
/// The placeholder for the result. Appended only on success.
@@ -750,6 +799,17 @@ class KUDU_EXPORT KuduClient : public sp::enable_shared_from_this<KuduClient> {
Status ListTables(std::vector<std::string>* tables,
const std::string& filter = "");
+ /// List soft-deleted tables only those names pass a substring
+ /// with names matching the specified @c filter.
+ ///
+ /// @param [out] tables
+ /// The placeholder for the result. Appended only on success.
+ /// @param [in] filter
+ /// Substring filter to use; empty sub-string filter matches all tables.
+ /// @return Status object for the operation.
+ Status ListSoftDeletedTables(std::vector<std::string>* tables,
+ const std::string& filter = "");
+
/// Check if the table given by 'table_name' exists.
///
/// @param [in] table_name
diff --git a/src/kudu/client/master_proxy_rpc.cc b/src/kudu/client/master_proxy_rpc.cc
index 8b8a2b814..6df6810f6 100644
--- a/src/kudu/client/master_proxy_rpc.cc
+++ b/src/kudu/client/master_proxy_rpc.cc
@@ -79,6 +79,8 @@ using master::ListTabletServersResponsePB;
using master::MasterServiceProxy;
using master::MasterErrorPB;
using master::MasterFeatures_Name;
+using master::RecallDeletedTableRequestPB;
+using master::RecallDeletedTableResponsePB;
using master::RemoveMasterRequestPB;
using master::RemoveMasterResponsePB;
using master::ReplaceTabletRequestPB;
@@ -305,6 +307,7 @@ template class AsyncLeaderMasterRpc<ListMastersRequestPB, ListMastersResponsePB>
template class AsyncLeaderMasterRpc<ListTablesRequestPB, ListTablesResponsePB>;
template class AsyncLeaderMasterRpc<ListTabletServersRequestPB, ListTabletServersResponsePB>;
template class AsyncLeaderMasterRpc<RemoveMasterRequestPB, RemoveMasterResponsePB>;
+template class AsyncLeaderMasterRpc<RecallDeletedTableRequestPB, RecallDeletedTableResponsePB>;
template class AsyncLeaderMasterRpc<ReplaceTabletRequestPB, ReplaceTabletResponsePB>;
} // namespace internal
diff --git a/src/kudu/client/master_proxy_rpc.h b/src/kudu/client/master_proxy_rpc.h
index 22548fbb5..d654ec8b3 100644
--- a/src/kudu/client/master_proxy_rpc.h
+++ b/src/kudu/client/master_proxy_rpc.h
@@ -21,6 +21,7 @@
#include <string>
#include <vector>
+#include "kudu/master/master.pb.h"
#include "kudu/rpc/response_callback.h"
#include "kudu/rpc/rpc.h"
#include "kudu/rpc/rpc_controller.h"
diff --git a/src/kudu/common/wire_protocol.cc b/src/kudu/common/wire_protocol.cc
index aa10d9a0c..c82a85049 100644
--- a/src/kudu/common/wire_protocol.cc
+++ b/src/kudu/common/wire_protocol.cc
@@ -684,10 +684,6 @@ Status ParseBoolConfig(const string& name, const string& value, bool* result) {
return Status::OK();
}
-static const std::string kTableHistoryMaxAgeSec = "kudu.table.history_max_age_sec";
-static const std::string kTableMaintenancePriority = "kudu.table.maintenance_priority";
-static const std::string kTableDisableCompaction = "kudu.table.disable_compaction";
-
Status ExtraConfigPBFromPBMap(const Map<string, string>& configs, TableExtraConfigPB* pb) {
static const unordered_set<string> kSupportedConfigs({kTableHistoryMaxAgeSec,
kTableMaintenancePriority,
diff --git a/src/kudu/common/wire_protocol.h b/src/kudu/common/wire_protocol.h
index e96af6ba2..737727bc2 100644
--- a/src/kudu/common/wire_protocol.h
+++ b/src/kudu/common/wire_protocol.h
@@ -57,6 +57,10 @@ class ServerEntryPB;
class ServerRegistrationPB;
class TableExtraConfigPB;
+static const std::string kTableHistoryMaxAgeSec = "kudu.table.history_max_age_sec";
+static const std::string kTableMaintenancePriority = "kudu.table.maintenance_priority";
+static const std::string kTableDisableCompaction = "kudu.table.disable_compaction";
+
// Convert the given C++ Status object into the equivalent Protobuf.
void StatusToPB(const Status& status, AppStatusPB* pb);
diff --git a/src/kudu/integration-tests/master_hms-itest.cc b/src/kudu/integration-tests/master_hms-itest.cc
index c48ef9bf7..9b989c879 100644
--- a/src/kudu/integration-tests/master_hms-itest.cc
+++ b/src/kudu/integration-tests/master_hms-itest.cc
@@ -22,9 +22,11 @@
#include <memory>
#include <optional>
#include <string>
+#include <type_traits>
#include <utility>
#include <vector>
+#include <gflags/gflags_declare.h>
#include <glog/stl_logging.h>
#include <gtest/gtest.h>
@@ -36,6 +38,7 @@
#include "kudu/gutil/map-util.h"
#include "kudu/gutil/strings/substitute.h"
#include "kudu/hms/hive_metastore_types.h"
+#include "kudu/hms/hms_catalog.h"
#include "kudu/hms/hms_client.h"
#include "kudu/integration-tests/external_mini_cluster-itest-base.h"
#include "kudu/integration-tests/hms_itest-base.h"
@@ -49,6 +52,8 @@
#include "kudu/util/test_macros.h"
#include "kudu/util/test_util.h"
+DECLARE_string(hive_metastore_uris);
+
using kudu::client::KuduTable;
using kudu::client::KuduTableAlterer;
using kudu::client::sp::shared_ptr;
@@ -476,6 +481,28 @@ TEST_F(MasterHmsTest, TestDeleteTable) {
ASSERT_OK(harness_.hms_client()->DropTable("default", "externalTable"));
}
+TEST_F(MasterHmsTest, TestSoftDeleteTable) {
+ // TODO(kedeng) : change the test case when state sync to HMS
+ // Create a Kudu table, then soft delete it from Kudu.
+ ASSERT_OK(CreateKuduTable("default", "a"));
+ NO_FATALS(CheckTable("default", "a", /*user=*/ nullopt));
+ hive::Table hms_table;
+ // Soft-delete related functions is not supported when HMS is enabled.
+ // We set hive_metastore_uris for hack HMS enable.
+ FLAGS_hive_metastore_uris = "thrift://127.0.0.1:0";
+ ASSERT_TRUE(hms::HmsCatalog::IsEnabled());
+ Status s = client_->SoftDeleteTable("default.a", 6000);
+ ASSERT_TRUE(s.IsNotSupported()) << s.ToString();
+ // The RecallTable is not supported, so the table id is ineffective.
+ s = client_->RecallTable("default.a");
+ ASSERT_TRUE(s.IsNotSupported()) << s.ToString();
+ ASSERT_STR_CONTAINS(s.ToString(), "RecallDeletedTable is not supported");
+ ASSERT_OK(harness_.hms_client()->GetTable("default", "a", &hms_table));
+ // The table is remain in the Kudu cluster.
+ shared_ptr<KuduTable> table;
+ ASSERT_OK(client_->OpenTable("default.a", &table));
+}
+
TEST_F(MasterHmsTest, TestNotificationLogListener) {
// Create a Kudu table.
ASSERT_OK(CreateKuduTable("default", "a"));
diff --git a/src/kudu/master/catalog_manager.cc b/src/kudu/master/catalog_manager.cc
index 7476df1a9..0772be7ff 100644
--- a/src/kudu/master/catalog_manager.cc
+++ b/src/kudu/master/catalog_manager.cc
@@ -65,7 +65,6 @@
#include <glog/logging.h>
#include <google/protobuf/arena.h>
#include <google/protobuf/stubs/common.h>
-#include <google/protobuf/stubs/port.h>
#include "kudu/cfile/type_encodings.h"
#include "kudu/common/common.pb.h"
@@ -79,7 +78,7 @@
#include "kudu/common/wire_protocol.h"
#include "kudu/common/wire_protocol.pb.h"
#include "kudu/consensus/consensus.pb.h"
-#include "kudu/consensus/consensus.proxy.h"
+#include "kudu/consensus/consensus.proxy.h" // IWYU pragma: keep
#include "kudu/consensus/opid_util.h"
#include "kudu/consensus/quorum_util.h"
#include "kudu/fs/fs_manager.h"
@@ -112,7 +111,7 @@
#include "kudu/master/table_metrics.h"
#include "kudu/master/ts_descriptor.h"
#include "kudu/master/ts_manager.h"
-#include "kudu/rpc/messenger.h"
+#include "kudu/rpc/messenger.h" // IWYU pragma: keep
#include "kudu/rpc/remote_user.h"
#include "kudu/rpc/rpc_context.h"
#include "kudu/rpc/rpc_controller.h"
@@ -122,13 +121,13 @@
#include "kudu/security/token.pb.h"
#include "kudu/security/token_signer.h"
#include "kudu/security/token_signing_key.h"
-#include "kudu/security/token_verifier.h"
+#include "kudu/security/token_verifier.h" // IWYU pragma: keep
#include "kudu/server/monitored_task.h"
#include "kudu/tablet/metadata.pb.h"
#include "kudu/tablet/ops/op_tracker.h"
#include "kudu/tablet/tablet_replica.h"
#include "kudu/tserver/tserver_admin.pb.h"
-#include "kudu/tserver/tserver_admin.proxy.h"
+#include "kudu/tserver/tserver_admin.proxy.h" // IWYU pragma: keep
#include "kudu/util/cache_metrics.h"
#include "kudu/util/condition_variable.h"
#include "kudu/util/debug/trace_event.h"
@@ -426,6 +425,7 @@ using kudu::consensus::RaftPeerPB;
using kudu::consensus::StartTabletCopyRequestPB;
using kudu::consensus::kMinimumTerm;
using kudu::hms::HmsClientVerifyKuduSyncConfig;
+using kudu::master::TableIdentifierPB;
using kudu::pb_util::SecureDebugString;
using kudu::pb_util::SecureShortDebugString;
using kudu::rpc::RpcContext;
@@ -595,6 +595,20 @@ class TableLoader : public TableVisitor {
Substitute("$0 or $1 [id=$2]", (*existing)->ToString(), l.data().name(), table_id));
}
}
+ // If the table is soft-deleted, add it into the soft-deleted map.
+ bool is_soft_deleted = l.mutable_data()->is_soft_deleted();
+ if (is_soft_deleted) {
+ auto* existing = InsertOrReturnExisting(&catalog_manager_->soft_deleted_table_names_map_,
+ CatalogManager::NormalizeTableName(l.data().name()),
+ table);
+ if (existing) {
+ return Status::IllegalState(
+ "when the Hive Metastore integration is enabled, Kudu soft-deleted table names must "
+ "not differ only by case; restart the master(s) with the Hive Metastore integration "
+ "disabled and rename one of the conflicting tables",
+ Substitute("$0 or $1 [id=$2]", (*existing)->ToString(), l.data().name(), table_id));
+ }
+ }
l.Commit();
if (!is_deleted) {
@@ -1554,6 +1568,7 @@ Status CatalogManager::VisitTablesAndTabletsUnlocked() {
// Clear the existing state.
normalized_table_names_map_.clear();
+ soft_deleted_table_names_map_.clear();
table_ids_map_.clear();
tablet_map_.clear();
@@ -1860,8 +1875,8 @@ Status CatalogManager::CreateTable(const CreateTableRequestPB* orig_req,
Schema client_schema;
RETURN_NOT_OK(SchemaFromPB(req.schema(), &client_schema));
- RETURN_NOT_OK(SetupError(
- ValidateClientSchema(normalized_table_name, req.owner(), req.comment(), client_schema),
+ RETURN_NOT_OK(SetupError(ValidateClientSchema(
+ normalized_table_name, req.owner(), req.comment(), client_schema),
resp, MasterErrorPB::INVALID_SCHEMA));
if (client_schema.has_column_ids()) {
return SetupError(Status::InvalidArgument("user requests should not have Column IDs"),
@@ -1982,7 +1997,7 @@ Status CatalogManager::CreateTable(const CreateTableRequestPB* orig_req,
TRACE("Acquired catalog manager lock");
// b. Verify that the table does not exist.
- table = FindPtrOrNull(normalized_table_names_map_, normalized_table_name);
+ table = FindTableWithNameUnlocked(normalized_table_name);
if (table != nullptr) {
return SetupError(Status::AlreadyPresent(Substitute(
"table $0 already exists with id $1", normalized_table_name, table->id())),
@@ -2144,8 +2159,8 @@ Status CatalogManager::IsCreateTableDone(const IsCreateTableDoneRequestPB* req,
username == owner),
resp, MasterErrorPB::NOT_AUTHORIZED);
};
- RETURN_NOT_OK(FindLockAndAuthorizeTable(
- *req, resp, LockMode::READ, authz_func, user, &table, &l));
+ RETURN_NOT_OK(FindLockAndAuthorizeTable(*req, resp, LockMode::READ, authz_func, user,
+ &table, &l, kNormalTableType));
RETURN_NOT_OK(CheckIfTableDeletedOrNotRunning(&l, resp));
// 2. Verify if the create is in-progress
@@ -2221,6 +2236,26 @@ scoped_refptr<TabletInfo> CatalogManager::CreateTabletInfo(
return tablet;
}
+scoped_refptr<TableInfo> CatalogManager::FindTableWithNameUnlocked(
+ const string& table_name,
+ TableInfoMapType map_type) {
+ scoped_refptr<TableInfo> normal_table(FindPtrOrNull(normalized_table_names_map_,
+ NormalizeTableName(table_name)));
+ scoped_refptr<TableInfo> soft_deleted_table(FindPtrOrNull(soft_deleted_table_names_map_,
+ NormalizeTableName(table_name)));
+
+ if (map_type == TableInfoMapType::kAllTableType) {
+ return normal_table ? normal_table : soft_deleted_table;
+ }
+ if (map_type == TableInfoMapType::kNormalTableType) {
+ return normal_table;
+ }
+ if (map_type == TableInfoMapType::kSoftDeletedTableType) {
+ return soft_deleted_table;
+ }
+ return nullptr;
+}
+
template<typename ReqClass, typename RespClass, typename F>
Status CatalogManager::FindLockAndAuthorizeTable(
const ReqClass& request,
@@ -2229,7 +2264,8 @@ Status CatalogManager::FindLockAndAuthorizeTable(
F authz_func,
const optional<string>& user,
scoped_refptr<TableInfo>* table_info,
- TableMetadataLock* table_lock) {
+ TableMetadataLock* table_lock,
+ TableInfoMapType map_type) {
TRACE("Looking up, locking, and authorizing table");
const TableIdentifierPB& table_identifier = request.table();
@@ -2271,15 +2307,14 @@ Status CatalogManager::FindLockAndAuthorizeTable(
// If the request contains both a table ID and table name, ensure that
// both match the same table.
- auto table_by_name = FindPtrOrNull(normalized_table_names_map_,
- NormalizeTableName(table_identifier.table_name()));
+ scoped_refptr<TableInfo> table_by_name =
+ FindTableWithNameUnlocked(table_identifier.table_name(), map_type);
if (table_identifier.has_table_name() &&
table.get() != table_by_name.get()) {
table_with_mismatched_name.swap(table_by_name);
}
} else if (table_identifier.has_table_name()) {
- table = FindPtrOrNull(normalized_table_names_map_,
- NormalizeTableName(table_identifier.table_name()));
+ table = FindTableWithNameUnlocked(table_identifier.table_name(), map_type);
} else {
return SetupError(Status::InvalidArgument("missing table ID or table name"),
response, MasterErrorPB::UNKNOWN_ERROR);
@@ -2342,6 +2377,117 @@ Status CatalogManager::FindLockAndAuthorizeTable(
return Status::OK();
}
+Status CatalogManager::SoftDeleteTableRpc(const DeleteTableRequestPB& req,
+ DeleteTableResponsePB* resp,
+ rpc::RpcContext* rpc) {
+ LOG(INFO) << Substitute("Servicing SoftDeleteTable request from $0:\n$1",
+ RequestorString(rpc), SecureShortDebugString(req));
+
+ bool is_soft_deleted_table = false;
+ bool is_expired_table = false;
+ Status s = GetTableStates(req.table(), kAllTableType, &is_soft_deleted_table, &is_expired_table);
+ if (s.ok() && is_soft_deleted_table && req.reserve_seconds() != 0) {
+ return SetupError(
+ Status::InvalidArgument(Substitute("soft_deleted table $0 should not be deleted",
+ req.table().table_name())),
+ resp, MasterErrorPB::TABLE_SOFT_DELETED);
+ }
+
+ // Reserve seconds equal 0 means delete it directly.
+ if (req.reserve_seconds() == 0) {
+ return DeleteTableRpc(req, resp, rpc);
+ }
+
+ DCHECK(!is_soft_deleted_table);
+ return SoftDeleteTable(req, resp, rpc);
+}
+
+Status CatalogManager::SoftDeleteTable(const DeleteTableRequestPB& req,
+ DeleteTableResponsePB* resp,
+ rpc::RpcContext* rpc) {
+ leader_lock_.AssertAcquiredForReading();
+
+ // TODO(kedeng) : soft_deleted state need sync to HMS.
+ // We disable soft-delete related functions when HMS is enabled.
+ if (hms::HmsCatalog::IsEnabled()) {
+ return SetupError(Status::NotSupported("SoftDeleteTable is not supported when HMS is enabled."),
+ resp, MasterErrorPB::UNKNOWN_ERROR);
+ }
+
+ optional<string> user;
+ if (rpc) {
+ user.emplace(rpc->remote_user().username());
+ }
+
+ // 1. Look up the table, lock it, and then check that the user is authorized
+ // to operate on the table. Last, mark it as soft_deleted.
+ scoped_refptr<TableInfo> table;
+ TableMetadataLock l;
+ auto authz_func = [&] (const string& username, const string& table_name, const string& owner) {
+ return SetupError(authz_provider_->AuthorizeDropTable(table_name, username, username == owner),
+ resp, MasterErrorPB::NOT_AUTHORIZED);
+ };
+ RETURN_NOT_OK(FindLockAndAuthorizeTable(req, resp, LockMode::WRITE, authz_func, user,
+ &table, &l));
+ if (l.data().is_deleted()) {
+ return SetupError(Status::NotFound("the table was deleted", l.data().pb.state_msg()),
+ resp, MasterErrorPB::TABLE_NOT_FOUND);
+ }
+
+ TRACE("Soft delete modifying in-memory table state");
+ string deletion_msg = "Table soft deleted at " + LocalTimeAsString();
+ // soft delete state change
+ l.mutable_data()->set_state(SysTablesEntryPB::SOFT_DELETED, deletion_msg);
+ l.mutable_data()->set_delete_timestamp(WallTime_Now());
+ l.mutable_data()->set_soft_deleted_reserved_seconds(req.reserve_seconds());
+
+ // 2. Look up the tablets, lock them, and mark them as soft deleted.
+ {
+ TRACE("Locking tablets");
+ vector<scoped_refptr<TabletInfo>> tablets;
+ TabletMetadataGroupLock lock(LockMode::RELEASED);
+ table->GetAllTablets(&tablets);
+ lock.AddMutableInfos(tablets);
+ lock.Lock(LockMode::WRITE);
+
+ for (const auto& t : tablets) {
+ t->mutable_metadata()->mutable_dirty()->set_state(
+ SysTabletsEntryPB::SOFT_DELETED, deletion_msg);
+ }
+
+ // 3. Update sys-catalog with the removed table and tablet state.
+ TRACE("Updating table and tablets from system table");
+ {
+ SysCatalogTable::Actions actions;
+ actions.table_to_update = table;
+ actions.tablets_to_update.assign(tablets.begin(), tablets.end());
+ Status s = sys_catalog_->Write(std::move(actions));
+ if (PREDICT_FALSE(!s.ok())) {
+ s = s.CloneAndPrepend("an error occurred while updating the sys-catalog");
+ LOG(WARNING) << s.ToString();
+ CheckIfNoLongerLeaderAndSetupError(s, resp);
+ return s;
+ }
+ }
+
+ // 4. Move the table from normal map to soft_deleted map.
+ {
+ TRACE("Moving table from normal map to soft_deleted map");
+ RETURN_NOT_OK(MoveToSoftDeletedContainer(req));
+ }
+
+ // 5. Commit the dirty tablet state.
+ lock.Commit();
+ }
+
+ // 6. Commit the dirty table state.
+ TRACE("Committing in-memory state");
+ l.Commit();
+
+ VLOG(1) << "Soft deleted table " << table->ToString();
+ return Status::OK();
+}
+
Status CatalogManager::DeleteTableRpc(const DeleteTableRequestPB& req,
DeleteTableResponsePB* resp,
rpc::RpcContext* rpc) {
@@ -2457,6 +2603,7 @@ Status CatalogManager::DeleteTable(const DeleteTableRequestPB& req,
string deletion_msg = "Table deleted at " + TimestampAsString(timestamp);
l.mutable_data()->set_state(SysTablesEntryPB::REMOVED, deletion_msg);
l.mutable_data()->pb.set_delete_timestamp(timestamp);
+ l.mutable_data()->pb.set_soft_deleted_reserved_seconds(req.reserve_seconds());
// 2. Look up the tablets, lock them, and mark them as deleted.
{
@@ -2494,7 +2641,8 @@ Status CatalogManager::DeleteTable(const DeleteTableRequestPB& req,
{
TRACE("Removing table from by-name map");
std::lock_guard<LockType> l_map(lock_);
- if (normalized_table_names_map_.erase(NormalizeTableName(l.data().name())) != 1) {
+ if ((normalized_table_names_map_.erase(NormalizeTableName(l.data().name())) != 1) &&
+ (soft_deleted_table_names_map_.erase(NormalizeTableName(l.data().name())) != 1)) {
LOG(FATAL) << "Could not remove table " << table->ToString()
<< " from map in response to DeleteTable request: "
<< SecureShortDebugString(req);
@@ -2526,6 +2674,117 @@ Status CatalogManager::DeleteTable(const DeleteTableRequestPB& req,
return Status::OK();
}
+Status CatalogManager::RecallDeletedTableRpc(const RecallDeletedTableRequestPB& req,
+ RecallDeletedTableResponsePB* resp,
+ rpc::RpcContext* rpc) {
+ LOG(INFO) << Substitute("Servicing RecallDeletedTableRpc request from $0:\n$1",
+ RequestorString(rpc), SecureShortDebugString(req));
+ RETURN_NOT_OK(RecallDeletedTable(req, resp, rpc));
+
+ if (req.has_new_table_name()) {
+ AlterTableRequestPB alter_req;
+ alter_req.mutable_table()->CopyFrom(req.table());
+ alter_req.set_new_table_name(req.new_table_name());
+
+ AlterTableResponsePB alter_resp;
+ Status s = AlterTableRpc(alter_req, &alter_resp, rpc);
+ if (!s.ok()) {
+ s = s.CloneAndPrepend("an error occurred while renaming the recalled table.");
+ LOG(WARNING) << s.ToString();
+ return s;
+ }
+ }
+
+ return Status::OK();
+}
+
+Status CatalogManager::RecallDeletedTable(const RecallDeletedTableRequestPB& req,
+ RecallDeletedTableResponsePB* resp,
+ rpc::RpcContext* rpc) {
+ bool is_soft_deleted_table = false;
+ bool is_expired_table = false;
+ Status s = GetTableStates(req.table(), kAllTableType, &is_soft_deleted_table, &is_expired_table);
+ if (s.ok() && !(is_soft_deleted_table || is_expired_table)) {
+ return SetupError(Status::NotFound(Substitute(
+ "the table $0 soft-deleted state $1, expired state $2, can't recall",
+ req.table().table_id(), is_soft_deleted_table, is_expired_table)),
+ resp, MasterErrorPB::TABLE_NOT_FOUND);
+ }
+
+ leader_lock_.AssertAcquiredForReading();
+
+ // TODO(kedeng) : normal state need sync to HMS
+ optional<string> user;
+ if (rpc) {
+ user.emplace(rpc->remote_user().username());
+ }
+
+ // 1. Look up the table, lock it, and then check that the user is authorized
+ // to operate on the table. Last, mark it as normal.
+ scoped_refptr<TableInfo> table;
+ TableMetadataLock l;
+ auto authz_func = [&] (const string& username, const string& table_name, const string& owner) {
+ return SetupError(authz_provider_->AuthorizeDropTable(table_name, username, username == owner),
+ resp, MasterErrorPB::NOT_AUTHORIZED);
+ };
+ RETURN_NOT_OK(FindLockAndAuthorizeTable(req, resp, LockMode::WRITE, authz_func, user,
+ &table, &l, kSoftDeletedTableType));
+
+ TRACE("Recall delete table modifying in-memory table state");
+ const time_t timestamp = time(nullptr);
+ string recalled_msg = "Table recalled at " + TimestampAsString(timestamp);
+ l.mutable_data()->set_state(SysTablesEntryPB::RUNNING, recalled_msg);
+ l.mutable_data()->set_delete_timestamp(0);
+ l.mutable_data()->set_soft_deleted_reserved_seconds(UINT32_MAX);
+
+ // 2. Look up the tablets, lock them, and mark them as normal.
+ {
+ TRACE("Locking tablets");
+ vector<scoped_refptr<TabletInfo>> tablets;
+ TabletMetadataGroupLock lock(LockMode::RELEASED);
+ table->GetAllTablets(&tablets);
+ lock.AddMutableInfos(tablets);
+ lock.Lock(LockMode::WRITE);
+
+ for (const auto& t : tablets) {
+ t->mutable_metadata()->mutable_dirty()->set_state(
+ SysTabletsEntryPB::RUNNING, recalled_msg);
+ t->mutable_metadata()->mutable_dirty()->pb.set_delete_timestamp(0);
+ }
+
+ // 3. Update sys-catalog with the recalled table and tablet state.
+ TRACE("Updating table and tablets from system table");
+ {
+ SysCatalogTable::Actions actions;
+ actions.table_to_update = table;
+ actions.tablets_to_update.assign(tablets.begin(), tablets.end());
+ s = sys_catalog_->Write(std::move(actions));
+ if (PREDICT_FALSE(!s.ok())) {
+ s = s.CloneAndPrepend("an error occurred while updating the sys-catalog");
+ LOG(WARNING) << s.ToString();
+ CheckIfNoLongerLeaderAndSetupError(s, resp);
+ return s;
+ }
+ }
+
+ // 4. Remove the table from soft_deleted map to normal map.
+ {
+ TRACE("Moving table from soft_deleted map to normal map");
+ RETURN_NOT_OK(MoveToNormalContainer(req));
+ }
+
+ // 5. Commit the dirty tablet state.
+ lock.Commit();
+ }
+
+ // 6. Commit the dirty table state.
+ TRACE("Committing in-memory state");
+ l.Commit();
+
+ VLOG(1) << "Recall deleted table " << req.table().table_name();
+ return Status::OK();
+}
+
Status CatalogManager::ApplyAlterSchemaSteps(
const SysTablesEntryPB& current_pb,
const vector<AlterTableRequestPB::Step>& steps,
@@ -2914,6 +3173,20 @@ Status CatalogManager::ApplyAlterPartitioningSteps(
Status CatalogManager::AlterTableRpc(const AlterTableRequestPB& req,
AlterTableResponsePB* resp,
rpc::RpcContext* rpc) {
+ LOG(INFO) << Substitute("Servicing AlterTable request from $0:\n$1",
+ RequestorString(rpc), SecureShortDebugString(req));
+
+ bool is_soft_deleted_table = false;
+ bool is_expired_table = false;
+ Status s = GetTableStates(req.table(), kAllTableType, &is_soft_deleted_table, &is_expired_table);
+ // Alter soft_deleted table is not allowed.
+ if (s.ok() && is_soft_deleted_table) {
+ return SetupError(
+ Status::InvalidArgument(Substitute("soft_deleted table $0 should not be altered",
+ req.table().table_name())),
+ resp, MasterErrorPB::TABLE_SOFT_DELETED);
+ }
+
leader_lock_.AssertAcquiredForReading();
if (req.modify_external_catalogs()) {
@@ -2923,10 +3196,7 @@ Status CatalogManager::AlterTableRpc(const AlterTableRequestPB& req,
RETURN_NOT_OK(WaitForNotificationLogListenerCatchUp(resp, rpc));
}
- LOG(INFO) << Substitute("Servicing AlterTable request from $0:\n$1",
- RequestorString(rpc), SecureShortDebugString(req));
-
- optional<string> user;
+ optional<const string> user;
if (rpc) {
user.emplace(rpc->remote_user().username());
}
@@ -2969,13 +3239,13 @@ Status CatalogManager::AlterTableRpc(const AlterTableRequestPB& req,
RETURN_NOT_OK(SchemaFromPB(l.data().pb.schema(), &schema));
// Rename the table in the HMS.
- auto s = hms_catalog_->AlterTable(table->id(),
- l.data().name(),
- normalized_new_table_name,
- GetClusterId(),
- l.data().owner(),
- schema,
- l.data().comment());
+ s = hms_catalog_->AlterTable(table->id(),
+ l.data().name(),
+ normalized_new_table_name,
+ GetClusterId(),
+ l.data().owner(),
+ schema,
+ l.data().comment());
if (PREDICT_TRUE(s.ok())) {
LOG(INFO) << Substitute("renamed table $0 in HMS: new name $1",
table->ToString(), normalized_new_table_name);
@@ -3199,7 +3469,6 @@ Status CatalogManager::AlterTable(const AlterTableRequestPB& req,
// 4. Validate and try to acquire the new table name.
string normalized_new_table_name = NormalizeTableName(req.new_table_name());
if (req.has_new_table_name()) {
-
// Validate the new table name.
RETURN_NOT_OK(SetupError(
ValidateIdentifier(req.new_table_name()).CloneAndPrepend("invalid table name"),
@@ -3504,7 +3773,7 @@ Status CatalogManager::IsAlterTableDone(const IsAlterTableDoneRequestPB* req,
resp, MasterErrorPB::NOT_AUTHORIZED);
};
RETURN_NOT_OK(FindLockAndAuthorizeTable(
- *req, resp, LockMode::READ, authz_func, user, &table, &l));
+ *req, resp, LockMode::READ, authz_func, user, &table, &l, kNormalTableType));
RETURN_NOT_OK(CheckIfTableDeletedOrNotRunning(&l, resp));
// 2. Verify if the alter is in-progress
@@ -3518,7 +3787,8 @@ Status CatalogManager::IsAlterTableDone(const IsAlterTableDoneRequestPB* req,
Status CatalogManager::GetTableSchema(const GetTableSchemaRequestPB* req,
GetTableSchemaResponsePB* resp,
const optional<string>& user,
- const TokenSigner* token_signer) {
+ const TokenSigner* token_signer,
+ TableInfoMapType map_type) {
leader_lock_.AssertAcquiredForReading();
// Lookup the table, verify if it exists, and then check that
@@ -3531,8 +3801,8 @@ Status CatalogManager::GetTableSchema(const GetTableSchemaRequestPB* req,
username == owner),
resp, MasterErrorPB::NOT_AUTHORIZED);
};
- RETURN_NOT_OK(FindLockAndAuthorizeTable(
- *req, resp, LockMode::READ, authz_func, user, &table, &l));
+ RETURN_NOT_OK(FindLockAndAuthorizeTable(*req, resp, LockMode::READ, authz_func, user,
+ &table, &l, map_type));
RETURN_NOT_OK(CheckIfTableDeletedOrNotRunning(&l, resp));
// If fully_applied_schema is set, use it, since an alter is in progress.
@@ -3576,9 +3846,19 @@ Status CatalogManager::ListTables(const ListTablesRequestPB* req,
vector<scoped_refptr<TableInfo>> tables_info;
{
+ bool show_soft_deleted = false;
+ if (req->has_show_soft_deleted()) {
+ show_soft_deleted = req->show_soft_deleted();
+ }
shared_lock<LockType> l(lock_);
- for (const TableInfoMap::value_type &entry : normalized_table_names_map_) {
- tables_info.emplace_back(entry.second);
+ if (show_soft_deleted) {
+ for (const auto& entry : soft_deleted_table_names_map_) {
+ tables_info.emplace_back(entry.second);
+ }
+ } else {
+ for (const auto& entry : normalized_table_names_map_) {
+ tables_info.emplace_back(entry.second);
+ }
}
}
unordered_set<int> table_types;
@@ -3726,7 +4006,7 @@ Status CatalogManager::GetTableStatistics(const GetTableStatisticsRequestPB* req
}
bool CatalogManager::IsTableWriteDisabled(const scoped_refptr<TableInfo>& table,
- const std::string& table_name) {
+ const string& table_name) {
uint64_t table_disk_size = 0;
uint64_t table_rows = 0;
if (table->GetMetrics()->TableSupportsOnDiskSize()) {
@@ -3802,7 +4082,8 @@ Status CatalogManager::TableNameExists(const string& table_name, bool* exists) {
leader_lock_.AssertAcquiredForReading();
shared_lock<LockType> l(lock_);
- *exists = ContainsKey(normalized_table_names_map_, NormalizeTableName(table_name));
+ scoped_refptr<TableInfo> table = FindTableWithNameUnlocked(table_name);
+ *exists = (table != nullptr);
return Status::OK();
}
@@ -6314,7 +6595,7 @@ void CatalogManager::ResetTableLocationsCache() {
}
Status CatalogManager::InitiateMasterChangeConfig(ChangeConfigOp op, const HostPort& hp,
- const std::string& uuid, rpc::RpcContext* rpc) {
+ const string& uuid, rpc::RpcContext* rpc) {
auto consensus = master_consensus();
if (!consensus) {
return Status::IllegalState("Consensus not running");
@@ -6387,6 +6668,85 @@ int CatalogManager::TSInfosDict::LookupOrAdd(const string& uuid,
});
}
+Status CatalogManager::MoveToSoftDeletedContainer(const DeleteTableRequestPB& req) {
+ TRACE("Moving table from normalized table map to soft_deleted table map.");
+
+ const string table_name = req.table().table_name();;
+ std::lock_guard<LockType> l_map(lock_);
+ auto table = FindPtrOrNull(normalized_table_names_map_,
+ NormalizeTableName(table_name));
+ if (!table) {
+ return Status::Corruption(Substitute("Table $0 is not exist in normal table map.",
+ table_name));
+ }
+
+ if (normalized_table_names_map_.erase(NormalizeTableName(table_name)) != 1) {
+ return Status::Corruption(Substitute("Could not move normal table $0 to soft_deleted map",
+ table_name));
+ }
+
+ DCHECK(!soft_deleted_table_names_map_[table_name]);
+ soft_deleted_table_names_map_[table_name] = table;
+ return Status::OK();
+}
+
+Status CatalogManager::MoveToNormalContainer(const RecallDeletedTableRequestPB& req) {
+ TRACE("Moving table from soft_deleted table map to normalized table map.");
+
+ std::lock_guard<LockType> l_map(lock_);
+ auto table = FindPtrOrNull(table_ids_map_, req.table().table_id());
+ if (!table) {
+ return Status::Corruption(Substitute("Table id $0 is not exist in soft_deleted table map.",
+ req.table().table_id()));
+ }
+
+ const string table_name = table->table_name();
+ if (soft_deleted_table_names_map_.erase(NormalizeTableName(table_name)) != 1) {
+ return Status::Corruption(Substitute("Could not move soft_deleted table $0 to normal map",
+ table_name));
+ }
+ DCHECK(!normalized_table_names_map_[table_name]);
+ normalized_table_names_map_[table_name] = table;
+
+ return Status::OK();
+}
+
+Status CatalogManager::GetTableStates(const TableIdentifierPB& table_identifier,
+ TableInfoMapType map_type,
+ bool* is_soft_deleted_table,
+ bool* is_expired_table) {
+ scoped_refptr<TableInfo> table_info;
+ *is_soft_deleted_table = false;
+ // Confirm the table really exists in the system catalog.
+ shared_lock<LockType> l(lock_);
+ scoped_refptr<TableInfo> table_by_name;
+ scoped_refptr<TableInfo> table_by_id;
+ if (table_identifier.has_table_name()) {
+ table_by_name = FindTableWithNameUnlocked(table_identifier.table_name(), map_type);
+ }
+ if (table_identifier.has_table_id()) {
+ table_by_id = FindPtrOrNull(table_ids_map_, table_identifier.table_id());
+ }
+
+ bool found = table_by_name || table_by_id;
+ bool table_unique = (table_identifier.has_table_name() && table_identifier.has_table_id())
+ ? (table_by_name == table_by_id) : true;
+ if (!table_unique || !found) {
+ // This function can only verify non HMS managed tables.
+ // If the table are not found by this, may exist in HMS, so we return directly.
+ // And subsequent functions will go to HMS for confirmation.
+ return Status::NotFound("table not found");
+ }
+ table_info = table_by_name ? table_by_name : table_by_id;
+
+ {
+ TableMetadataLock table_l(table_info.get(), LockMode::READ);
+ *is_soft_deleted_table = table_info->metadata().state().is_soft_deleted();
+ *is_expired_table = table_info->metadata().state().is_expired();
+ }
+
+ return Status::OK();
+}
////////////////////////////////////////////////////////////
// CatalogManager::ScopedLeaderSharedLock
////////////////////////////////////////////////////////////
@@ -6499,6 +6859,7 @@ INITTED_AND_LEADER_OR_RESPOND(GetTableLocationsResponsePB);
INITTED_AND_LEADER_OR_RESPOND(GetTableSchemaResponsePB);
INITTED_AND_LEADER_OR_RESPOND(GetTableStatisticsResponsePB);
INITTED_AND_LEADER_OR_RESPOND(GetTabletLocationsResponsePB);
+INITTED_AND_LEADER_OR_RESPOND(RecallDeletedTableResponsePB);
INITTED_AND_LEADER_OR_RESPOND(RemoveMasterResponsePB);
INITTED_AND_LEADER_OR_RESPOND(ReplaceTabletResponsePB);
@@ -6622,6 +6983,11 @@ string TableInfo::ToString() const {
return Substitute("$0 [id=$1]", l.data().pb.name(), table_id_);
}
+string TableInfo::table_name() const {
+ TableMetadataLock l(this, LockMode::READ);
+ return l.data().pb.name();
+}
+
uint32_t TableInfo::schema_version() const {
TableMetadataLock l(this, LockMode::READ);
return l.data().pb.version();
diff --git a/src/kudu/master/catalog_manager.h b/src/kudu/master/catalog_manager.h
index 9282fa28b..340d3996d 100644
--- a/src/kudu/master/catalog_manager.h
+++ b/src/kudu/master/catalog_manager.h
@@ -41,10 +41,12 @@
#include "kudu/common/partition.h"
#include "kudu/consensus/metadata.pb.h"
#include "kudu/consensus/raft_consensus.h"
+#include "kudu/gutil/integral_types.h"
#include "kudu/gutil/macros.h"
#include "kudu/gutil/port.h"
#include "kudu/gutil/ref_counted.h"
#include "kudu/gutil/strings/stringpiece.h"
+#include "kudu/gutil/walltime.h"
#include "kudu/master/master.pb.h"
#include "kudu/tablet/metadata.pb.h"
#include "kudu/tserver/tablet_replica_lookup.h"
@@ -62,7 +64,6 @@ namespace protobuf {
class Arena;
} // namespace protobuf
} // namespace google
-template <class X> struct GoodFastHash;
namespace kudu {
@@ -81,6 +82,7 @@ struct ColumnId;
// Working around FRIEND_TEST() ugliness.
namespace client {
+class ClientTest_TestSoftDeleteAndReserveTable_Test;
class ServiceUnavailableRetryClientTest_CreateTable_Test;
} // namespace client
@@ -131,7 +133,8 @@ struct TableMetrics;
// It wraps the underlying protobuf to add useful accessors.
struct PersistentTabletInfo {
bool is_running() const {
- return pb.state() == SysTabletsEntryPB::RUNNING;
+ return pb.state() == SysTabletsEntryPB::RUNNING ||
+ pb.state() == SysTabletsEntryPB::SOFT_DELETED;
}
bool is_deleted() const {
@@ -139,6 +142,10 @@ struct PersistentTabletInfo {
pb.state() == SysTabletsEntryPB::DELETED;
}
+ bool is_soft_deleted() const {
+ return pb.state() == SysTabletsEntryPB::SOFT_DELETED;
+ }
+
// Helper to set the state of the tablet with a custom message.
// Requires that the caller has prepared this object for write.
// The change will only be visible after Commit().
@@ -244,7 +251,24 @@ struct PersistentTableInfo {
bool is_running() const {
return pb.state() == SysTablesEntryPB::RUNNING ||
- pb.state() == SysTablesEntryPB::ALTERING;
+ pb.state() == SysTablesEntryPB::ALTERING ||
+ pb.state() == SysTablesEntryPB::SOFT_DELETED;
+ }
+
+ bool is_soft_deleted() const {
+ return pb.state() == SysTablesEntryPB::SOFT_DELETED;
+ }
+
+ // Expired table must in SOFT_DELETED state.
+ bool is_expired() const {
+ if (!is_soft_deleted() ||
+ !pb.has_delete_timestamp() ||
+ !pb.has_soft_deleted_reserved_seconds()) {
+ return false;
+ }
+
+ return pb.delete_timestamp() + pb.soft_deleted_reserved_seconds()
+ <= static_cast<uint64_t>(WallTime_Now());
}
// Return the table's name.
@@ -262,9 +286,19 @@ struct PersistentTableInfo {
return pb.comment();
}
- // Helper to set the state of the tablet with a custom message.
+ // Helper to set the state of the table with a custom message.
void set_state(SysTablesEntryPB::State state, const std::string& msg);
+ // Helper to set the delete_timestamp of the table.
+ void set_delete_timestamp(int64 timestamp) {
+ pb.set_delete_timestamp(timestamp);
+ }
+
+ // Helper to set the soft_deleted_reserved_seconds of the table.
+ void set_soft_deleted_reserved_seconds(uint32 reserve_seconds) {
+ pb.set_soft_deleted_reserved_seconds(reserve_seconds);
+ }
+
SysTablesEntryPB pb;
};
@@ -291,6 +325,9 @@ class TableInfo : public RefCountedThreadSafe<TableInfo> {
explicit TableInfo(std::string table_id);
std::string ToString() const;
+
+ std::string table_name() const;
+
uint32_t schema_version() const;
// Return the table's ID. Does not require synchronization.
@@ -600,6 +637,12 @@ class CatalogManager : public tserver::TabletReplicaLookupIf {
void Shutdown();
+ enum TableInfoMapType {
+ kNormalTableType = 1 << 0, // normalized_table_names_map_
+ kSoftDeletedTableType = 1 << 1, // soft_deleted_table_names_map_
+ kAllTableType = 0b00000011
+ };
+
// Create a new Table with the specified attributes.
//
// The RPC context is provded for logging/tracing purposes,
@@ -622,12 +665,26 @@ class CatalogManager : public tserver::TabletReplicaLookupIf {
DeleteTableResponsePB* resp,
rpc::RpcContext* rpc) WARN_UNUSED_RESULT;
+ // Mark the table as soft-deleted with ability to restore it back within
+ // the soft-delete reservation period.
+ Status SoftDeleteTableRpc(const DeleteTableRequestPB& req,
+ DeleteTableResponsePB* resp,
+ rpc::RpcContext* rpc);
+
// Delete the specified table in response to a 'DROP TABLE' HMS notification
// log listener event.
Status DeleteTableHms(const std::string& table_name,
const std::string& table_id,
int64_t notification_log_event_id) WARN_UNUSED_RESULT;
+ // Recall a table in response to a RecallDeletedTableRequestPB RPC.
+ //
+ // The RPC context is provided for logging/tracing purposes,
+ // but this function does not itself respond to the RPC.
+ Status RecallDeletedTableRpc(const RecallDeletedTableRequestPB& req,
+ RecallDeletedTableResponsePB* resp,
+ rpc::RpcContext* rpc) WARN_UNUSED_RESULT;
+
// Alter the specified table in response to an AlterTableRequest RPC.
//
// The RPC context is provided for logging/tracing purposes,
@@ -658,7 +715,8 @@ class CatalogManager : public tserver::TabletReplicaLookupIf {
Status GetTableSchema(const GetTableSchemaRequestPB* req,
GetTableSchemaResponsePB* resp,
const std::optional<std::string>& user,
- const security::TokenSigner* token_signer);
+ const security::TokenSigner* token_signer,
+ TableInfoMapType map_type = kAllTableType);
// Lists all the running tables. If 'user' is provided, only lists those that
// the given user is authorized to see.
@@ -842,8 +900,25 @@ class CatalogManager : public tserver::TabletReplicaLookupIf {
Status InitiateMasterChangeConfig(ChangeConfigOp op, const HostPort& hp,
const std::string& uuid, rpc::RpcContext* rpc);
+ // Check whether the reservation period for a soft-deleted table has expired.
+ Status GetTableStates(const TableIdentifierPB& table_identifier,
+ TableInfoMapType map_type,
+ bool* is_soft_deleted_table,
+ bool* is_expired_table = nullptr);
+
+ // Move the deleted table to soft-deleted map for keep it temporarily.
+ Status MoveToSoftDeletedContainer(const DeleteTableRequestPB& req);
+
+ // Move the soft-deleted table to normal map for recall.
+ Status MoveToNormalContainer(const RecallDeletedTableRequestPB& req);
+
+ // Use for seach table with table name.
+ scoped_refptr<TableInfo> FindTableWithNameUnlocked(const std::string& table_name,
+ TableInfoMapType map_type = kAllTableType);
+
private:
- // These tests call ElectedAsLeaderCb() directly.
+ // These tests calls ElectedAsLeaderCb() directly.
+ FRIEND_TEST(kudu::client::ClientTest, TestSoftDeleteAndReserveTable);
FRIEND_TEST(MasterTest, TestShutdownDuringTableVisit);
FRIEND_TEST(MasterTest, TestGetTableLocationsDuringRepeatedTableVisit);
FRIEND_TEST(kudu::AuthzTokenTest, TestSingleMasterUnavailable);
@@ -866,6 +941,21 @@ class CatalogManager : public tserver::TabletReplicaLookupIf {
void GetAllTabletsForTests(std::vector<scoped_refptr<TabletInfo>>* tablets);
+ // Soft delete the specified table and keep it for reserve time.
+ // If 'user' is provided, checks that the user is authorized to delete the table.
+ // Otherwise, it indicates its an internal operation (originates from catalog
+ // manager).
+ Status SoftDeleteTable(const DeleteTableRequestPB& req,
+ DeleteTableResponsePB* resp,
+ rpc::RpcContext* rpc);
+
+ // Recall the specified table. If the table is in soft-deleted state and within the
+ // reservation period, the operation make sense. Otherwise, the request will be
+ // rejected.
+ Status RecallDeletedTable(const RecallDeletedTableRequestPB& req,
+ RecallDeletedTableResponsePB* resp,
+ rpc::RpcContext* rpc);
+
// Check whether the table's write limit is reached,
// if true, the write permission should be disabled.
static bool IsTableWriteDisabled(const scoped_refptr<TableInfo>& table,
@@ -1049,7 +1139,8 @@ class CatalogManager : public tserver::TabletReplicaLookupIf {
F authz_func,
const std::optional<std::string>& user,
scoped_refptr<TableInfo>* table_info,
- TableMetadataLock* table_lock) WARN_UNUSED_RESULT;
+ TableMetadataLock* table_lock,
+ TableInfoMapType map_type = kAllTableType) WARN_UNUSED_RESULT;
// Extract the set of tablets that must be processed because not running yet.
void ExtractTabletsToProcess(std::vector<scoped_refptr<TabletInfo>>* tablets_to_process);
@@ -1199,6 +1290,8 @@ class CatalogManager : public tserver::TabletReplicaLookupIf {
// Table maps: table-id -> TableInfo and normalized-table-name -> TableInfo
TableInfoMap table_ids_map_;
TableInfoMap normalized_table_names_map_;
+ // Table maps: soft-deleted-table-name -> TableInfo
+ TableInfoMap soft_deleted_table_names_map_;
// Tablet maps: tablet-id -> TabletInfo
TabletInfoMap tablet_map_;
diff --git a/src/kudu/master/master-test.cc b/src/kudu/master/master-test.cc
index 3a5002be0..2b2d73505 100644
--- a/src/kudu/master/master-test.cc
+++ b/src/kudu/master/master-test.cc
@@ -30,6 +30,7 @@
#include <set>
#include <string>
#include <thread>
+#include <type_traits>
#include <unordered_map>
#include <unordered_set>
#include <utility>
@@ -54,6 +55,7 @@
#include "kudu/consensus/replica_management.pb.h"
#include "kudu/generated/version_defines.h"
#include "kudu/gutil/dynamic_annotations.h"
+#include "kudu/gutil/integral_types.h"
#include "kudu/gutil/map-util.h"
#include "kudu/gutil/ref_counted.h"
#include "kudu/gutil/strings/split.h"
@@ -68,7 +70,6 @@
#include "kudu/master/mini_master.h"
#include "kudu/master/sys_catalog.h"
#include "kudu/master/table_metrics.h"
-#include "kudu/master/ts_descriptor.h"
#include "kudu/master/ts_manager.h"
#include "kudu/rpc/messenger.h"
#include "kudu/rpc/rpc_controller.h"
@@ -98,6 +99,12 @@
#include "kudu/util/test_util.h"
#include "kudu/util/version_info.h"
+namespace kudu {
+namespace master {
+class TSDescriptor;
+} // namespace master
+} // namespace kudu
+
using kudu::consensus::ReplicaManagementInfoPB;
using kudu::itest::GetClusterId;
using kudu::pb_util::SecureDebugString;
@@ -222,6 +229,10 @@ class MasterTest : public KuduTest {
Status GetTablePartitionSchema(const string& table_name,
PartitionSchemaPB* ps_pb);
+ Status SoftDelete(const string& table_name, uint32 reserve_seconds);
+
+ Status RecallTable(const string& table_id);
+
shared_ptr<Messenger> client_messenger_;
unique_ptr<MiniMaster> mini_master_;
Master* master_;
@@ -343,6 +354,23 @@ Status MasterTest::GetTablePartitionSchema(const string& table_name,
return Status::OK();
}
+Status MasterTest::SoftDelete(const string& table_name, uint32 reserve_seconds) {
+ DeleteTableRequestPB req;
+ DeleteTableResponsePB resp;
+ RpcController controller;
+ req.mutable_table()->set_table_name(table_name);
+ req.set_reserve_seconds(reserve_seconds);
+ return master_->catalog_manager()->SoftDeleteTableRpc(req, &resp, nullptr);
+}
+
+Status MasterTest::RecallTable(const string& table_id) {
+ RecallDeletedTableRequestPB req;
+ RecallDeletedTableResponsePB resp;
+ RpcController controller;
+ req.mutable_table()->set_table_id(table_id);
+ return master_->catalog_manager()->RecallDeletedTableRpc(req, &resp, nullptr);
+}
+
void MasterTest::DoListTables(const ListTablesRequestPB& req, ListTablesResponsePB* resp) {
RpcController controller;
ASSERT_OK(proxy_->ListTables(req, resp, &controller));
@@ -3600,5 +3628,109 @@ TEST_F(MasterStartupTest, StartupWebPage) {
ASSERT_STR_CONTAINS(buf.ToString(), "\"start_rpc_server_status\":100");
}
+TEST_F(MasterTest, GetTableStatesWithName) {
+ const char* kTableName = "testtable";
+ const Schema kTableSchema({ ColumnSchema("key", INT32) }, 1);
+ bool is_soft_deleted_table = false;
+ bool is_expired_table = false;
+ TableIdentifierPB table_identifier;
+ table_identifier.set_table_name(kTableName);
+
+ // Create a new table.
+ ASSERT_OK(CreateTable(kTableName, kTableSchema));
+ ListTablesResponsePB tables;
+ NO_FATALS(DoListAllTables(&tables));
+ ASSERT_EQ(1, tables.tables_size());
+ ASSERT_EQ(kTableName, tables.tables(0).name());
+ string table_id = tables.tables(0).id();
+ ASSERT_FALSE(table_id.empty());
+
+ {
+ // Default table is not expired.
+ CatalogManager::ScopedLeaderSharedLock l(master_->catalog_manager());
+ ASSERT_OK(master_->catalog_manager()->GetTableStates(
+ table_identifier, CatalogManager::TableInfoMapType::kAllTableType,
+ &is_soft_deleted_table, &is_expired_table));
+ ASSERT_FALSE(is_soft_deleted_table);
+ ASSERT_FALSE(is_expired_table);
+ }
+
+ {
+ // In reserve time, table is not expired.
+ CatalogManager::ScopedLeaderSharedLock l(master_->catalog_manager());
+ ASSERT_OK(SoftDelete(kTableName, 100));
+ ASSERT_OK(master_->catalog_manager()->GetTableStates(
+ table_identifier, CatalogManager::TableInfoMapType::kAllTableType,
+ &is_soft_deleted_table, &is_expired_table));
+ ASSERT_TRUE(is_soft_deleted_table);
+ ASSERT_FALSE(is_expired_table);
+ ASSERT_OK(RecallTable(table_id));
+ }
+
+ {
+ // After reserve time, table is expired.
+ CatalogManager::ScopedLeaderSharedLock l(master_->catalog_manager());
+ ASSERT_OK(SoftDelete(kTableName, 1));
+ SleepFor(MonoDelta::FromSeconds(1));
+ ASSERT_OK(master_->catalog_manager()->GetTableStates(
+ table_identifier, CatalogManager::TableInfoMapType::kAllTableType,
+ &is_soft_deleted_table, &is_expired_table));
+ ASSERT_TRUE(is_soft_deleted_table);
+ ASSERT_TRUE(is_expired_table);
+ }
+}
+
+TEST_F(MasterTest, GetTableStatesWithId) {
+ const char* kTableName = "testtable";
+ const Schema kTableSchema({ ColumnSchema("key", INT32) }, 1);
+ bool is_soft_deleted_table = false;
+ bool is_expired_table = false;
+
+ // Create a new table.
+ ASSERT_OK(CreateTable(kTableName, kTableSchema));
+ ListTablesResponsePB tables;
+ NO_FATALS(DoListAllTables(&tables));
+ ASSERT_EQ(1, tables.tables_size());
+ ASSERT_EQ(kTableName, tables.tables(0).name());
+ string table_id = tables.tables(0).id();
+ ASSERT_FALSE(table_id.empty());
+ TableIdentifierPB table_identifier;
+ table_identifier.set_table_id(table_id);
+
+ {
+ // Default table is not expired.
+ CatalogManager::ScopedLeaderSharedLock l(master_->catalog_manager());
+ ASSERT_OK(master_->catalog_manager()->GetTableStates(
+ table_identifier, CatalogManager::TableInfoMapType::kAllTableType,
+ &is_soft_deleted_table, &is_expired_table));
+ ASSERT_FALSE(is_soft_deleted_table);
+ ASSERT_FALSE(is_expired_table);
+ }
+
+ {
+ // In reserve time, table is not expired.
+ CatalogManager::ScopedLeaderSharedLock l(master_->catalog_manager());
+ ASSERT_OK(SoftDelete(kTableName, 100));
+ ASSERT_OK(master_->catalog_manager()->GetTableStates(
+ table_identifier, CatalogManager::TableInfoMapType::kAllTableType,
+ &is_soft_deleted_table, &is_expired_table));
+ ASSERT_TRUE(is_soft_deleted_table);
+ ASSERT_FALSE(is_expired_table);
+ ASSERT_OK(RecallTable(table_id));
+ }
+
+ {
+ // After reserve time, table is expired.
+ CatalogManager::ScopedLeaderSharedLock l(master_->catalog_manager());
+ ASSERT_OK(SoftDelete(kTableName, 1));
+ SleepFor(MonoDelta::FromSeconds(1));
+ ASSERT_OK(master_->catalog_manager()->GetTableStates(
+ table_identifier, CatalogManager::TableInfoMapType::kAllTableType,
+ &is_soft_deleted_table, &is_expired_table));
+ ASSERT_TRUE(is_soft_deleted_table);
+ ASSERT_TRUE(is_expired_table);
+ }
+}
+
} // namespace master
} // namespace kudu
diff --git a/src/kudu/master/master.cc b/src/kudu/master/master.cc
index 6262a8490..d54a53732 100644
--- a/src/kudu/master/master.cc
+++ b/src/kudu/master/master.cc
@@ -20,8 +20,10 @@
#include <algorithm>
#include <functional>
#include <memory>
+#include <optional>
#include <ostream>
#include <string>
+#include <type_traits>
#include <vector>
#include <gflags/gflags.h>
@@ -32,12 +34,12 @@
#include "kudu/common/wire_protocol.h"
#include "kudu/common/wire_protocol.pb.h"
#include "kudu/consensus/metadata.pb.h"
-#include "kudu/consensus/raft_consensus.h"
#include "kudu/fs/error_manager.h"
#include "kudu/fs/fs_manager.h"
#include "kudu/gutil/ref_counted.h"
#include "kudu/gutil/strings/join.h"
#include "kudu/gutil/strings/substitute.h"
+#include "kudu/hms/hms_catalog.h"
#include "kudu/master/catalog_manager.h"
#include "kudu/master/location_cache.h"
#include "kudu/master/master.pb.h"
@@ -48,7 +50,6 @@
#include "kudu/master/ts_manager.h"
#include "kudu/master/txn_manager.h"
#include "kudu/master/txn_manager_service.h"
-#include "kudu/rpc/messenger.h"
#include "kudu/rpc/rpc_controller.h"
#include "kudu/rpc/service_if.h"
#include "kudu/security/token_signer.h"
@@ -57,6 +58,7 @@
#include "kudu/server/webserver.h"
#include "kudu/tserver/tablet_copy_service.h"
#include "kudu/tserver/tablet_service.h"
+#include "kudu/util/countdown_latch.h"
#include "kudu/util/flag_tags.h"
#include "kudu/util/logging.h"
#include "kudu/util/maintenance_manager.h"
@@ -64,10 +66,17 @@
#include "kudu/util/net/net_util.h"
#include "kudu/util/net/sockaddr.h"
#include "kudu/util/status.h"
+#include "kudu/util/thread.h"
#include "kudu/util/threadpool.h"
#include "kudu/util/timer.h"
#include "kudu/util/version_info.h"
+namespace kudu {
+namespace rpc {
+class Messenger;
+} // namespace rpc
+} // namespace kudu
+
DEFINE_int32(master_registration_rpc_timeout_ms, 1500,
"Timeout for retrieving master registration over RPC.");
TAG_FLAG(master_registration_rpc_timeout_ms, experimental);
@@ -90,6 +99,11 @@ DEFINE_int64(authz_token_validity_seconds, 60 * 5,
"validity period expires.");
TAG_FLAG(authz_token_validity_seconds, experimental);
+DEFINE_int32(check_expired_table_interval_seconds, 60,
+ "Interval (in seconds) to check whether there is any soft_deleted table "
+ "with expired reservation period. Such tables will be purged and cannot "
+ "be recalled after that.");
+
DEFINE_string(location_mapping_cmd, "",
"A Unix command which takes a single argument, the IP address or "
"hostname of a tablet server or client, and returns the location "
@@ -110,6 +124,7 @@ using kudu::transactions::TxnManagerServiceImpl;
using kudu::tserver::ConsensusServiceImpl;
using kudu::tserver::TabletCopyServiceImpl;
using std::min;
+using std::nullopt;
using std::shared_ptr;
using std::string;
using std::unique_ptr;
@@ -169,7 +184,7 @@ Status GetMasterEntryForHost(const shared_ptr<rpc::Messenger>& messenger,
}
return Status::OK();
}
-} // anonymous namespace
+} // anonymous namespace
Master::Master(const MasterOptions& opts)
: KuduServer("Master", opts, "kudu.master"),
@@ -271,6 +286,10 @@ Status Master::StartAsync() {
RETURN_NOT_OK(ScheduleTxnManagerInit());
}
+ // Soft-delete related functions are not supported when HMS is enabled.
+ if (!hms::HmsCatalog::IsEnabled()) {
+ RETURN_NOT_OK(StartExpiredReservedTablesDeleterThread());
+ }
state_ = kRunning;
return Status::OK();
@@ -429,6 +448,61 @@ void Master::ShutdownImpl() {
state_ = kStopped;
}
+Status Master::StartExpiredReservedTablesDeleterThread() {
+ return Thread::Create("master",
+ "expired-reserved-tables-deleter",
+ [this]() { this->ExpiredReservedTablesDeleterThread(); },
+ &expired_reserved_tables_deleter_thread_);
+}
+
+void Master::ExpiredReservedTablesDeleterThread() {
+ // How often to attempt to delete expired tables.
+ const MonoDelta kWait = MonoDelta::FromSeconds(FLAGS_check_expired_table_interval_seconds);
+ while (!stop_background_threads_latch_.WaitFor(kWait)) {
+ WARN_NOT_OK(DeleteExpiredReservedTables(), "Unable to delete expired reserved tables");
+ }
+}
+
+Status Master::DeleteExpiredReservedTables() {
+ CatalogManager::ScopedLeaderSharedLock l(catalog_manager());
+ if (!l.first_failed_status().ok()) {
+ // Skip checking if this master is not leader.
+ return Status::OK();
+ }
+
+ ListTablesRequestPB list_req;
+ ListTablesResponsePB list_resp;
+ // Set for soft_deleted table map.
+ list_req.set_show_soft_deleted(true);
+ RETURN_NOT_OK(catalog_manager_->ListTables(&list_req, &list_resp, nullopt));
+ for (const auto& table : list_resp.tables()) {
+ bool is_soft_deleted_table = false;
+ bool is_expired_table = false;
+ TableIdentifierPB table_identifier;
+ table_identifier.set_table_name(table.name());
+ Status s = catalog_manager_->GetTableStates(
+ table_identifier,
+ CatalogManager::TableInfoMapType::kSoftDeletedTableType,
+ &is_soft_deleted_table, &is_expired_table);
+ if (s.ok() && (!is_soft_deleted_table || !is_expired_table)) {
+ continue;
+ }
+
+ // Delete the table.
+ DeleteTableRequestPB del_req;
+ del_req.mutable_table()->set_table_id(table.id());
+ del_req.mutable_table()->set_table_name(table.name());
+ del_req.set_reserve_seconds(0);
+ DeleteTableResponsePB del_resp;
+ LOG(INFO) << "Start to delete soft_deleted table " << table.name();
+ WARN_NOT_OK(catalog_manager_->DeleteTableRpc(del_req, &del_resp, nullptr),
+ Substitute("Failed to delete soft_deleted table $0 (table_id=$1)",
+ table.name(), table.id()));
+ }
+
+ return Status::OK();
+}
+
Status Master::ListMasters(vector<ServerEntryPB>* masters) const {
auto consensus = catalog_manager_->master_consensus();
if (!consensus) {
diff --git a/src/kudu/master/master.h b/src/kudu/master/master.h
index 46ed76c94..dd451a908 100644
--- a/src/kudu/master/master.h
+++ b/src/kudu/master/master.h
@@ -25,6 +25,7 @@
#include "kudu/common/wire_protocol.pb.h"
#include "kudu/gutil/macros.h"
#include "kudu/gutil/port.h"
+#include "kudu/gutil/ref_counted.h"
#include "kudu/kserver/kserver.h"
#include "kudu/master/master_options.h"
#include "kudu/util/promise.h"
@@ -36,6 +37,7 @@ class HostPort;
class MaintenanceManager;
class MonoDelta;
class MonoTime;
+class Thread;
class ThreadPool;
namespace rpc {
@@ -163,6 +165,11 @@ class Master : public kserver::KuduServer {
// safe in a particular case.
void ShutdownImpl();
+ // Start thread to purge soft-deleted tables with expired reservations.
+ Status StartExpiredReservedTablesDeleterThread();
+ void ExpiredReservedTablesDeleterThread();
+ Status DeleteExpiredReservedTables();
+
enum MasterState {
kStopped,
kInitialized,
@@ -205,6 +212,8 @@ class Master : public kserver::KuduServer {
std::unique_ptr<TSManager> ts_manager_;
+ scoped_refptr<Thread> expired_reserved_tables_deleter_thread_;
+
DISALLOW_COPY_AND_ASSIGN(Master);
};
diff --git a/src/kudu/master/master.proto b/src/kudu/master/master.proto
index 13050c30f..6a7cd9f16 100644
--- a/src/kudu/master/master.proto
+++ b/src/kudu/master/master.proto
@@ -89,6 +89,9 @@ message MasterErrorPB {
// Master is already part of the Raft configuration.
MASTER_ALREADY_PRESENT = 15;
+
+ // The requested table is in soft_deleted state.
+ TABLE_SOFT_DELETED = 16;
}
// The error code.
@@ -129,6 +132,7 @@ message SysTabletsEntryPB {
RUNNING = 2;
REPLACED = 3;
DELETED = 4;
+ SOFT_DELETED = 5;
}
// DEPRECATED. Replaced by 'partition'.
@@ -165,6 +169,7 @@ message SysTablesEntryPB {
RUNNING = 2;
ALTERING = 3;
REMOVED = 4;
+ SOFT_DELETED = 5;
}
// Table name
@@ -221,6 +226,9 @@ message SysTablesEntryPB {
// The delete time of the table, in seconds since the epoch.
optional int64 delete_timestamp = 18;
+
+ // The reservation time interval (in seconds) between soft delete and delete.
+ optional uint32 soft_deleted_reserved_seconds = 19;
}
// The on-disk entry in the sys.catalog table ("metadata" column) to represent
@@ -576,6 +584,9 @@ message DeleteTableRequestPB {
// Whether to apply the deletion to external catalogs, such as the Hive Metastore,
// which the Kudu master has been configured to integrate with.
optional bool modify_external_catalogs = 2 [default = true];
+
+ // Reserve seconds after the table has been deleted.
+ optional uint32 reserve_seconds = 3;
}
message DeleteTableResponsePB {
@@ -583,6 +594,19 @@ message DeleteTableResponsePB {
optional MasterErrorPB error = 1;
}
+message RecallDeletedTableRequestPB {
+ required TableIdentifierPB table = 1;
+
+ // If this field is set, that's the name for the recalled table.
+ // Otherwise, the recalled table will use the original table name.
+ optional string new_table_name = 2;
+}
+
+message RecallDeletedTableResponsePB {
+ // The error, if an error occurred with this request.
+ optional MasterErrorPB error = 1;
+}
+
message ListTablesRequestPB {
// When used, only returns tables that satisfy a substring match on name_filter.
optional string name_filter = 1;
@@ -595,6 +619,11 @@ message ListTablesRequestPB {
// Set this field 'true' to include information on the partition backed by
// each tablet in the result list.
optional bool list_tablet_with_partition = 3 [default = false];
+
+ // Use to select the tables type for display.
+ // Only show regular tables if false.
+ // Only show soft_deleted tables if true.
+ optional bool show_soft_deleted = 4;
}
message ListTablesResponsePB {
@@ -1176,6 +1205,10 @@ service MasterService {
option (kudu.rpc.authz_method) = "AuthorizeClient";
}
+ rpc RecallDeletedTable(RecallDeletedTableRequestPB) returns (RecallDeletedTableResponsePB) {
+ option (kudu.rpc.authz_method) = "AuthorizeClient";
+ }
+
rpc AlterTable(AlterTableRequestPB) returns (AlterTableResponsePB) {
option (kudu.rpc.authz_method) = "AuthorizeClientOrServiceUser";
}
diff --git a/src/kudu/master/master_service.cc b/src/kudu/master/master_service.cc
index e2eaa3cbb..9db93a038 100644
--- a/src/kudu/master/master_service.cc
+++ b/src/kudu/master/master_service.cc
@@ -572,7 +572,30 @@ void MasterServiceImpl::DeleteTable(const DeleteTableRequestPB* req,
return;
}
- Status s = server_->catalog_manager()->DeleteTableRpc(*req, resp, rpc);
+ Status s = server_->catalog_manager()->SoftDeleteTableRpc(*req, resp, rpc);
+ CheckRespErrorOrSetUnknown(s, resp);
+ rpc->RespondSuccess();
+}
+
+void MasterServiceImpl::RecallDeletedTable(const RecallDeletedTableRequestPB* req,
+ RecallDeletedTableResponsePB* resp,
+ rpc::RpcContext* rpc) {
+ CatalogManager::ScopedLeaderSharedLock l(server_->catalog_manager());
+ if (!l.CheckIsInitializedAndIsLeaderOrRespond(resp, rpc)) {
+ return;
+ }
+
+ // Soft-delete related functions is not supported when HMS is enabled.
+ if (hms::HmsCatalog::IsEnabled()) {
+ StatusToPB(Status::NotSupported("RecallDeletedTable is not supported when HMS is enabled."),
+ resp->mutable_error()->mutable_status());
+ resp->mutable_error()->set_code(MasterErrorPB::UNKNOWN_ERROR);
+ rpc->RespondSuccess();
+ return;
+ }
+
+ Status s = server_->catalog_manager()->RecallDeletedTableRpc(
+ *req, resp, rpc);
CheckRespErrorOrSetUnknown(s, resp);
rpc->RespondSuccess();
}
diff --git a/src/kudu/master/master_service.h b/src/kudu/master/master_service.h
index f86a225d3..3eaf6c86d 100644
--- a/src/kudu/master/master_service.h
+++ b/src/kudu/master/master_service.h
@@ -71,6 +71,8 @@ class ListTabletServersResponsePB;
class Master;
class PingRequestPB;
class PingResponsePB;
+class RecallDeletedTableRequestPB;
+class RecallDeletedTableResponsePB;
class RefreshAuthzCacheRequestPB;
class RefreshAuthzCacheResponsePB;
class RemoveMasterRequestPB;
@@ -144,6 +146,10 @@ class MasterServiceImpl : public MasterServiceIf {
DeleteTableResponsePB* resp,
rpc::RpcContext* rpc) override;
+ void RecallDeletedTable(const RecallDeletedTableRequestPB* req,
+ RecallDeletedTableResponsePB* resp,
+ rpc::RpcContext* rpc) override;
+
void AlterTable(const AlterTableRequestPB* req,
AlterTableResponsePB* resp,
rpc::RpcContext* rpc) override;
diff --git a/src/kudu/server/server_base.cc b/src/kudu/server/server_base.cc
index b49d8efb7..b3aee0960 100644
--- a/src/kudu/server/server_base.cc
+++ b/src/kudu/server/server_base.cc
@@ -499,12 +499,12 @@ ServerBase::ServerBase(string name, const ServerBaseOptions& options,
result_tracker_(new rpc::ResultTracker(shared_ptr<MemTracker>(
MemTracker::CreateTracker(-1, "result-tracker", mem_tracker_)))),
is_first_run_(false),
+ stop_background_threads_latch_(1),
dns_resolver_(new DnsResolver(
FLAGS_dns_resolver_max_threads_num,
FLAGS_dns_resolver_cache_capacity_mb * 1024 * 1024,
MonoDelta::FromSeconds(FLAGS_dns_resolver_cache_ttl_sec))),
- options_(options),
- stop_background_threads_latch_(1) {
+ options_(options) {
metric_entity_->NeverRetire(
METRIC_merged_entities_count_of_server.InstantiateHidden(metric_entity_, 1));
diff --git a/src/kudu/server/server_base.h b/src/kudu/server/server_base.h
index 42cc2d627..3802af1ac 100644
--- a/src/kudu/server/server_base.h
+++ b/src/kudu/server/server_base.h
@@ -19,12 +19,13 @@
#include <cstdint>
#include <memory>
#include <string>
+#include <type_traits>
#include <glog/logging.h>
#include "kudu/gutil/macros.h"
#include "kudu/gutil/ref_counted.h"
-#include "kudu/rpc/messenger.h"
+#include "kudu/rpc/messenger.h" // IWYU pragma: keep
#include "kudu/security/simple_acl.h"
#include "kudu/server/server_base_options.h"
#include "kudu/util/countdown_latch.h"
@@ -222,6 +223,8 @@ class ServerBase {
// The ACL of users who may act as part of the Kudu service.
security::SimpleAcl service_acl_;
+ CountDownLatch stop_background_threads_latch_;
+
private:
Status InitAcls();
void GenerateInstanceID();
@@ -273,7 +276,6 @@ class ServerBase {
#ifdef TCMALLOC_ENABLED
scoped_refptr<Thread> tcmalloc_memory_gc_thread_;
#endif
- CountDownLatch stop_background_threads_latch_;
std::unique_ptr<ScopedGLogMetrics> glog_metrics_;
diff --git a/src/kudu/tools/kudu-admin-test.cc b/src/kudu/tools/kudu-admin-test.cc
index 1480ce6f3..088a36e75 100644
--- a/src/kudu/tools/kudu-admin-test.cc
+++ b/src/kudu/tools/kudu-admin-test.cc
@@ -26,6 +26,7 @@
#include <ostream>
#include <string>
#include <tuple>
+#include <type_traits>
#include <unordered_map>
#include <unordered_set>
#include <utility>
@@ -1651,7 +1652,8 @@ TEST_F(AdminCliTest, TestDeleteTable) {
"table",
"delete",
master_address,
- kTableId
+ kTableId,
+ "-reserve_seconds=0"
);
vector<string> tables;
@@ -1665,16 +1667,51 @@ TEST_F(AdminCliTest, TestListTables) {
NO_FATALS(BuildAndStart());
- string stdout;
- ASSERT_OK(RunKuduTool({
- "table",
- "list",
- cluster_->master()->bound_rpc_addr().ToString()
- }, &stdout));
+ {
+ string stdout;
+ ASSERT_OK(RunKuduTool({
+ "table",
+ "list",
+ cluster_->master()->bound_rpc_addr().ToString()
+ }, &stdout));
+
+ vector<string> stdout_lines = Split(stdout, ",", strings::SkipEmpty());
+ ASSERT_EQ(1, stdout_lines.size());
+ ASSERT_EQ(Substitute("$0\n", kTableId), stdout_lines[0]);
+ }
- vector<string> stdout_lines = Split(stdout, ",", strings::SkipEmpty());
- ASSERT_EQ(1, stdout_lines.size());
- ASSERT_EQ(Substitute("$0\n", kTableId), stdout_lines[0]);
+ {
+ string stdout;
+ ASSERT_OK(RunKuduTool({
+ "table",
+ "list",
+ "-soft_deleted_only=true",
+ cluster_->master()->bound_rpc_addr().ToString()
+ }, &stdout));
+
+ vector<string> stdout_lines = Split(stdout, ",", strings::SkipEmpty());
+ ASSERT_EQ(0, stdout_lines.size());
+
+ ASSERT_OK(RunKuduTool({
+ "table",
+ "delete",
+ cluster_->master()->bound_rpc_addr().ToString(),
+ kTableId
+ }, &stdout));
+
+ stdout.clear();
+ ASSERT_OK(RunKuduTool({
+ "table",
+ "list",
+ "-soft_deleted_only=true",
+ cluster_->master()->bound_rpc_addr().ToString()
+ }, &stdout));
+
+ stdout_lines.clear();
+ stdout_lines = Split(stdout, ",", strings::SkipEmpty());
+ ASSERT_EQ(1, stdout_lines.size());
+ ASSERT_EQ(Substitute("$0\n", kTableId), stdout_lines[0]);
+ }
}
TEST_F(AdminCliTest, TestListTablesDetail) {
diff --git a/src/kudu/tools/kudu-tool-test.cc b/src/kudu/tools/kudu-tool-test.cc
index c668b648c..064f1cad4 100644
--- a/src/kudu/tools/kudu-tool-test.cc
+++ b/src/kudu/tools/kudu-tool-test.cc
@@ -1425,6 +1425,7 @@ TEST_F(ToolTest, TestModeHelp) {
"get_extra_configs.*Get the extra configuration properties for a table",
"list.*List tables",
"locate_row.*Locate which tablet a row belongs to",
+ "recall.*Recall a deleted but still reserved table",
"rename_column.*Rename a column",
"rename_table.*Rename a table",
"scan.*Scan rows from a table",
@@ -4546,8 +4547,9 @@ TEST_F(ToolTest, TestDeleteTable) {
ASSERT_EQ(exist, true);
// Delete the table.
- NO_FATALS(RunActionStdoutNone(Substitute("table delete $0 $1 --nomodify_external_catalogs",
- master_addr, kTableName)));
+ NO_FATALS(RunActionStdoutNone(Substitute(
+ "table delete $0 $1 --nomodify_external_catalogs -reserve_seconds=0",
+ master_addr, kTableName)));
// Check that the table does not exist.
ASSERT_OK(client->TableExists(kTableName, &exist));
@@ -4582,6 +4584,57 @@ TEST_F(ToolTest, TestRenameTable) {
ASSERT_OK(client->OpenTable(kTableName, &table));
}
+TEST_F(ToolTest, TestRecallTable) {
+ NO_FATALS(StartExternalMiniCluster());
+ shared_ptr<KuduClient> client;
+ ASSERT_OK(cluster_->CreateClient(nullptr, &client));
+ string master_addr = cluster_->master()->bound_rpc_addr().ToString();
+
+ constexpr const char* const kTableName = "kudu.table";
+
+ // Create the table.
+ TestWorkload workload(cluster_.get());
+ workload.set_table_name(kTableName);
+ workload.set_num_replicas(1);
+ workload.Setup();
+
+ shared_ptr<KuduTable> table;
+ ASSERT_OK(client->OpenTable(kTableName, &table));
+ string table_id = table->id();
+
+ // Delete the table.
+ string out;
+ NO_FATALS(RunActionStdoutNone(Substitute("table delete $0 $1",
+ master_addr, kTableName)));
+
+ // List soft_deleted table.
+ vector<string> kudu_tables;
+ ASSERT_OK(client->ListSoftDeletedTables(&kudu_tables));
+ ASSERT_EQ(1, kudu_tables.size());
+ kudu_tables.clear();
+ ASSERT_OK(client->ListTables(&kudu_tables));
+ ASSERT_EQ(0, kudu_tables.size());
+
+ // Can't create a table with the same name whose state is in soft_deleted.
+ workload.set_table_name(kTableName);
+ workload.set_num_replicas(1);
+ NO_FATALS(workload.Setup());
+ ASSERT_OK(client->ListTables(&kudu_tables));
+ ASSERT_EQ(0, kudu_tables.size());
+
+ const string kNewTableName = "kudu.table.new";
+ // Try to recall the soft_deleted table with new name.
+ NO_FATALS(RunActionStdoutNone(Substitute("table recall $0 $1 --new_table_name=$2",
+ master_addr, table_id, kNewTableName)));
+
+ ASSERT_OK(client->ListTables(&kudu_tables));
+ ASSERT_EQ(1, kudu_tables.size());
+ ASSERT_TRUE(kNewTableName == kudu_tables[0]);
+ kudu_tables.clear();
+ ASSERT_OK(client->ListSoftDeletedTables(&kudu_tables));
+ ASSERT_EQ(0, kudu_tables.size());
+}
+
TEST_F(ToolTest, TestRenameColumn) {
NO_FATALS(StartExternalMiniCluster());
constexpr const char* const kTableName = "table";
@@ -6115,7 +6168,7 @@ TEST_P(ToolTestKerberosParameterized, TestCheckAndAutomaticFixHmsMetadata) {
NO_FATALS(ValidateHmsEntries(&hms_client, kudu_client, "my_db", "table", master_addrs_str));
vector<string> kudu_tables;
- kudu_client->ListTables(&kudu_tables);
+ ASSERT_OK(kudu_client->ListTables(&kudu_tables));
std::sort(kudu_tables.begin(), kudu_tables.end());
ASSERT_EQ(vector<string>({
"default.bad_id",
@@ -6277,7 +6330,7 @@ TEST_P(ToolTestKerberosParameterized, TestCheckAndManualFixHmsMetadata) {
// Ensure the tables are available.
vector<string> kudu_tables;
- kudu_client->ListTables(&kudu_tables);
+ ASSERT_OK(kudu_client->ListTables(&kudu_tables));
std::sort(kudu_tables.begin(), kudu_tables.end());
ASSERT_EQ(vector<string>({
"default.conflicting_legacy_table",
diff --git a/src/kudu/tools/tool_action_table.cc b/src/kudu/tools/tool_action_table.cc
index 50ce04f3c..7bfe717bd 100644
--- a/src/kudu/tools/tool_action_table.cc
+++ b/src/kudu/tools/tool_action_table.cc
@@ -141,6 +141,11 @@ bool ValidateShowHashPartitionInfo() {
GROUP_FLAG_VALIDATOR(show_hash_partition_info, ValidateShowHashPartitionInfo);
+DEFINE_bool(soft_deleted_only, false,
+ "Show only soft-deleted tables if set true, otherwise show regular tables.");
+DEFINE_string(new_table_name, "",
+ "The new name for the recalled table. "
+ "Leave empty to recall the table under its original name.");
DEFINE_bool(modify_external_catalogs, true,
"Whether to modify external catalogs, such as the Hive Metastore, "
"when renaming or dropping a table.");
@@ -166,6 +171,8 @@ DEFINE_bool(show_avro_format_schema, false,
"Display the table schema in avro format. When enabled it only outputs the "
"table schema in Avro format without any other information like "
"partition/owner/comments. It cannot be used in conjunction with other flags");
+DEFINE_uint32(reserve_seconds, 604800,
+ "Reserve seconds before purging a soft-deleted table.");
DECLARE_bool(create_table);
DECLARE_int32(create_table_replication_factor);
@@ -234,8 +241,8 @@ class TableLister {
client.get(),
&tables_info,
"" /* filter */,
- FLAGS_show_tablet_partition_info));
-
+ FLAGS_show_tablet_partition_info,
+ FLAGS_soft_deleted_only));
vector<string> table_filters = Split(FLAGS_tables, ",", strings::SkipEmpty());
for (const auto& tinfo : tables_info) {
const auto& tname = tinfo.table_name;
@@ -474,7 +481,9 @@ Status DeleteTable(const RunnerContext& context) {
const string& table_name = FindOrDie(context.required_args, kTableNameArg);
client::sp::shared_ptr<KuduClient> client;
RETURN_NOT_OK(CreateKuduClient(context, &client));
- return client->DeleteTableInCatalogs(table_name, FLAGS_modify_external_catalogs);
+ return client->DeleteTableInCatalogs(table_name,
+ FLAGS_modify_external_catalogs,
+ FLAGS_reserve_seconds);
}
Status DescribeTable(const RunnerContext& context) {
@@ -731,6 +740,13 @@ Status SetRowCountLimit(const RunnerContext& context) {
return alterer->Alter();
}
+Status RecallTable(const RunnerContext& context) {
+ const string& table_id = FindOrDie(context.required_args, kTabletIdArg);
+ client::sp::shared_ptr<KuduClient> client;
+ RETURN_NOT_OK(CreateKuduClient(context, &client));
+ return client->RecallTable(table_id, FLAGS_new_table_name);
+}
+
Status RenameTable(const RunnerContext& context) {
const string& table_name = FindOrDie(context.required_args, kTableNameArg);
const string& new_table_name = FindOrDie(context.required_args, kNewTableNameArg);
@@ -808,8 +824,8 @@ Status SetExtraConfig(const RunnerContext& context) {
client::sp::shared_ptr<KuduClient> client;
RETURN_NOT_OK(CreateKuduClient(context, &client));
unique_ptr<KuduTableAlterer> alterer(client->NewTableAlterer(table_name));
- alterer->AlterExtraConfig({ { config_name, config_value} });
- return alterer->Alter();
+ return alterer->AlterExtraConfig({ { config_name, config_value} })
+ ->Alter();
}
Status GetExtraConfigs(const RunnerContext& context) {
@@ -1696,6 +1712,7 @@ unique_ptr<Mode> BuildTableMode() {
.Description("Delete a table")
.AddRequiredParameter({ kTableNameArg, "Name of the table to delete" })
.AddOptionalParameter("modify_external_catalogs")
+ .AddOptionalParameter("reserve_seconds")
.Build();
unique_ptr<Action> describe_table =
@@ -1709,6 +1726,7 @@ unique_ptr<Mode> BuildTableMode() {
unique_ptr<Action> list_tables =
ClusterActionBuilder("list", &ListTables)
.Description("List tables")
+ .AddOptionalParameter("soft_deleted_only")
.AddOptionalParameter("tables")
.AddOptionalParameter("list_tablets")
.AddOptionalParameter("show_tablet_partition_info")
@@ -1740,6 +1758,14 @@ unique_ptr<Mode> BuildTableMode() {
.AddRequiredParameter({ kNewColumnNameArg, "New column name" })
.Build();
+ unique_ptr<Action> recall =
+ ActionBuilder("recall", &RecallTable)
+ .Description("Recall a deleted but still reserved table")
+ .AddRequiredParameter({ kMasterAddressesArg, kMasterAddressesArgDesc })
+ .AddRequiredParameter({ kTabletIdArg, "ID of the table to recall" })
+ .AddOptionalParameter("new_table_name")
+ .Build();
+
unique_ptr<Action> rename_table =
ClusterActionBuilder("rename_table", &RenameTable)
.Description("Rename a table")
@@ -1961,6 +1987,7 @@ unique_ptr<Mode> BuildTableMode() {
.AddAction(std::move(get_extra_configs))
.AddAction(std::move(list_tables))
.AddAction(std::move(locate_row))
+ .AddAction(std::move(recall))
.AddAction(std::move(rename_column))
.AddAction(std::move(rename_table))
.AddAction(std::move(scan_table))