HBASE-25617 Revisit the span names (#2998)
[hbase.git] / hbase-client / src / main / java / org / apache / hadoop / hbase / client / AsyncAdmin.java
blob2ed624ca01f831d5dfc98609520ebdeab5755ab5
1 /*
2 * Licensed to the Apache Software Foundation (ASF) under one
3 * or more contributor license agreements. See the NOTICE file
4 * distributed with this work for additional information
5 * regarding copyright ownership. The ASF licenses this file
6 * to you under the Apache License, Version 2.0 (the
7 * "License"); you may not use this file except in compliance
8 * with the License. You may obtain a copy of the License at
10 * http://www.apache.org/licenses/LICENSE-2.0
12 * Unless required by applicable law or agreed to in writing, software
13 * distributed under the License is distributed on an "AS IS" BASIS,
14 * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
15 * See the License for the specific language governing permissions and
16 * limitations under the License.
18 package org.apache.hadoop.hbase.client;
20 import static org.apache.hadoop.hbase.util.FutureUtils.addListener;
22 import java.io.IOException;
23 import java.util.Arrays;
24 import java.util.Collection;
25 import java.util.EnumSet;
26 import java.util.HashMap;
27 import java.util.List;
28 import java.util.Map;
29 import java.util.Optional;
30 import java.util.Set;
31 import java.util.concurrent.CompletableFuture;
32 import java.util.function.Function;
33 import java.util.regex.Pattern;
34 import java.util.stream.Collectors;
35 import org.apache.hadoop.hbase.CacheEvictionStats;
36 import org.apache.hadoop.hbase.ClusterMetrics;
37 import org.apache.hadoop.hbase.ClusterMetrics.Option;
38 import org.apache.hadoop.hbase.NamespaceDescriptor;
39 import org.apache.hadoop.hbase.RegionMetrics;
40 import org.apache.hadoop.hbase.ServerName;
41 import org.apache.hadoop.hbase.TableName;
42 import org.apache.hadoop.hbase.client.replication.TableCFs;
43 import org.apache.hadoop.hbase.client.security.SecurityCapability;
44 import org.apache.hadoop.hbase.net.Address;
45 import org.apache.hadoop.hbase.quotas.QuotaFilter;
46 import org.apache.hadoop.hbase.quotas.QuotaSettings;
47 import org.apache.hadoop.hbase.quotas.SpaceQuotaSnapshotView;
48 import org.apache.hadoop.hbase.replication.ReplicationPeerConfig;
49 import org.apache.hadoop.hbase.replication.ReplicationPeerDescription;
50 import org.apache.hadoop.hbase.replication.SyncReplicationState;
51 import org.apache.hadoop.hbase.rsgroup.RSGroupInfo;
52 import org.apache.hadoop.hbase.security.access.GetUserPermissionsRequest;
53 import org.apache.hadoop.hbase.security.access.Permission;
54 import org.apache.hadoop.hbase.security.access.UserPermission;
55 import org.apache.hadoop.hbase.util.Pair;
56 import org.apache.yetus.audience.InterfaceAudience;
58 import org.apache.hbase.thirdparty.com.google.protobuf.RpcChannel;
60 /**
61 * The asynchronous administrative API for HBase.
62 * @since 2.0.0
64 @InterfaceAudience.Public
65 public interface AsyncAdmin {
67 /**
68 * @param tableName Table to check.
69 * @return True if table exists already. The return value will be wrapped by a
70 * {@link CompletableFuture}.
72 CompletableFuture<Boolean> tableExists(TableName tableName);
74 /**
75 * List all the userspace tables.
76 * @return - returns a list of TableDescriptors wrapped by a {@link CompletableFuture}.
78 default CompletableFuture<List<TableDescriptor>> listTableDescriptors() {
79 return listTableDescriptors(false);
82 /**
83 * List all the tables.
84 * @param includeSysTables False to match only against userspace tables
85 * @return - returns a list of TableDescriptors wrapped by a {@link CompletableFuture}.
87 CompletableFuture<List<TableDescriptor>> listTableDescriptors(boolean includeSysTables);
89 /**
90 * List all the tables matching the given pattern.
91 * @param pattern The compiled regular expression to match against
92 * @param includeSysTables False to match only against userspace tables
93 * @return - returns a list of TableDescriptors wrapped by a {@link CompletableFuture}.
95 CompletableFuture<List<TableDescriptor>> listTableDescriptors(Pattern pattern,
96 boolean includeSysTables);
98 /**
99 * List specific tables including system tables.
100 * @param tableNames the table list to match against
101 * @return - returns a list of TableDescriptors wrapped by a {@link CompletableFuture}.
103 CompletableFuture<List<TableDescriptor>> listTableDescriptors(List<TableName> tableNames);
106 * Get list of table descriptors by namespace.
107 * @param name namespace name
108 * @return returns a list of TableDescriptors wrapped by a {@link CompletableFuture}.
110 CompletableFuture<List<TableDescriptor>> listTableDescriptorsByNamespace(String name);
113 * List all of the names of userspace tables.
114 * @return a list of table names wrapped by a {@link CompletableFuture}.
115 * @see #listTableNames(Pattern, boolean)
117 default CompletableFuture<List<TableName>> listTableNames() {
118 return listTableNames(false);
122 * List all of the names of tables.
123 * @param includeSysTables False to match only against userspace tables
124 * @return a list of table names wrapped by a {@link CompletableFuture}.
126 CompletableFuture<List<TableName>> listTableNames(boolean includeSysTables);
129 * List all of the names of userspace tables.
130 * @param pattern The regular expression to match against
131 * @param includeSysTables False to match only against userspace tables
132 * @return a list of table names wrapped by a {@link CompletableFuture}.
134 CompletableFuture<List<TableName>> listTableNames(Pattern pattern, boolean includeSysTables);
137 * Get list of table names by namespace.
138 * @param name namespace name
139 * @return The list of table names in the namespace wrapped by a {@link CompletableFuture}.
141 CompletableFuture<List<TableName>> listTableNamesByNamespace(String name);
144 * Method for getting the tableDescriptor
145 * @param tableName as a {@link TableName}
146 * @return the read-only tableDescriptor wrapped by a {@link CompletableFuture}.
148 CompletableFuture<TableDescriptor> getDescriptor(TableName tableName);
151 * Creates a new table.
152 * @param desc table descriptor for table
154 CompletableFuture<Void> createTable(TableDescriptor desc);
157 * Creates a new table with the specified number of regions. The start key specified will become
158 * the end key of the first region of the table, and the end key specified will become the start
159 * key of the last region of the table (the first region has a null start key and the last region
160 * has a null end key). BigInteger math will be used to divide the key range specified into enough
161 * segments to make the required number of total regions.
162 * @param desc table descriptor for table
163 * @param startKey beginning of key range
164 * @param endKey end of key range
165 * @param numRegions the total number of regions to create
167 CompletableFuture<Void> createTable(TableDescriptor desc, byte[] startKey, byte[] endKey,
168 int numRegions);
171 * Creates a new table with an initial set of empty regions defined by the specified split keys.
172 * The total number of regions created will be the number of split keys plus one.
173 * Note : Avoid passing empty split key.
174 * @param desc table descriptor for table
175 * @param splitKeys array of split keys for the initial regions of the table
177 CompletableFuture<Void> createTable(TableDescriptor desc, byte[][] splitKeys);
180 * Modify an existing table, more IRB friendly version.
181 * @param desc modified description of the table
183 CompletableFuture<Void> modifyTable(TableDescriptor desc);
186 * Deletes a table.
187 * @param tableName name of table to delete
189 CompletableFuture<Void> deleteTable(TableName tableName);
192 * Truncate a table.
193 * @param tableName name of table to truncate
194 * @param preserveSplits True if the splits should be preserved
196 CompletableFuture<Void> truncateTable(TableName tableName, boolean preserveSplits);
199 * Enable a table. The table has to be in disabled state for it to be enabled.
200 * @param tableName name of the table
202 CompletableFuture<Void> enableTable(TableName tableName);
205 * Disable a table. The table has to be in enabled state for it to be disabled.
206 * @param tableName
208 CompletableFuture<Void> disableTable(TableName tableName);
211 * @param tableName name of table to check
212 * @return true if table is on-line. The return value will be wrapped by a
213 * {@link CompletableFuture}.
215 CompletableFuture<Boolean> isTableEnabled(TableName tableName);
218 * @param tableName name of table to check
219 * @return true if table is off-line. The return value will be wrapped by a
220 * {@link CompletableFuture}.
222 CompletableFuture<Boolean> isTableDisabled(TableName tableName);
225 * @param tableName name of table to check
226 * @return true if all regions of the table are available. The return value will be wrapped by a
227 * {@link CompletableFuture}.
229 CompletableFuture<Boolean> isTableAvailable(TableName tableName);
232 * Add a column family to an existing table.
233 * @param tableName name of the table to add column family to
234 * @param columnFamily column family descriptor of column family to be added
236 CompletableFuture<Void> addColumnFamily(TableName tableName,
237 ColumnFamilyDescriptor columnFamily);
240 * Delete a column family from a table.
241 * @param tableName name of table
242 * @param columnFamily name of column family to be deleted
244 CompletableFuture<Void> deleteColumnFamily(TableName tableName, byte[] columnFamily);
247 * Modify an existing column family on a table.
248 * @param tableName name of table
249 * @param columnFamily new column family descriptor to use
251 CompletableFuture<Void> modifyColumnFamily(TableName tableName,
252 ColumnFamilyDescriptor columnFamily);
255 * Create a new namespace.
256 * @param descriptor descriptor which describes the new namespace
258 CompletableFuture<Void> createNamespace(NamespaceDescriptor descriptor);
261 * Modify an existing namespace.
262 * @param descriptor descriptor which describes the new namespace
264 CompletableFuture<Void> modifyNamespace(NamespaceDescriptor descriptor);
267 * Delete an existing namespace. Only empty namespaces (no tables) can be removed.
268 * @param name namespace name
270 CompletableFuture<Void> deleteNamespace(String name);
273 * Get a namespace descriptor by name
274 * @param name name of namespace descriptor
275 * @return A descriptor wrapped by a {@link CompletableFuture}.
277 CompletableFuture<NamespaceDescriptor> getNamespaceDescriptor(String name);
280 * List available namespaces
281 * @return List of namespaces wrapped by a {@link CompletableFuture}.
283 CompletableFuture<List<String>> listNamespaces();
286 * List available namespace descriptors
287 * @return List of descriptors wrapped by a {@link CompletableFuture}.
289 CompletableFuture<List<NamespaceDescriptor>> listNamespaceDescriptors();
292 * Get all the online regions on a region server.
294 CompletableFuture<List<RegionInfo>> getRegions(ServerName serverName);
297 * Get the regions of a given table.
299 CompletableFuture<List<RegionInfo>> getRegions(TableName tableName);
302 * Flush a table.
303 * @param tableName table to flush
305 CompletableFuture<Void> flush(TableName tableName);
308 * Flush the specified column family stores on all regions of the passed table.
309 * This runs as a synchronous operation.
310 * @param tableName table to flush
311 * @param columnFamily column family within a table
313 CompletableFuture<Void> flush(TableName tableName, byte[] columnFamily);
316 * Flush an individual region.
317 * @param regionName region to flush
319 CompletableFuture<Void> flushRegion(byte[] regionName);
322 * Flush a column family within a region.
323 * @param regionName region to flush
324 * @param columnFamily column family within a region. If not present, flush the region's all
325 * column families.
327 CompletableFuture<Void> flushRegion(byte[] regionName, byte[] columnFamily);
330 * Flush all region on the region server.
331 * @param serverName server to flush
333 CompletableFuture<Void> flushRegionServer(ServerName serverName);
336 * Compact a table. When the returned CompletableFuture is done, it only means the compact request
337 * was sent to HBase and may need some time to finish the compact operation.
338 * Throws {@link org.apache.hadoop.hbase.TableNotFoundException} if table not found.
339 * @param tableName table to compact
341 default CompletableFuture<Void> compact(TableName tableName) {
342 return compact(tableName, CompactType.NORMAL);
346 * Compact a column family within a table. When the returned CompletableFuture is done, it only
347 * means the compact request was sent to HBase and may need some time to finish the compact
348 * operation.
349 * Throws {@link org.apache.hadoop.hbase.TableNotFoundException} if table not found.
350 * @param tableName table to compact
351 * @param columnFamily column family within a table. If not present, compact the table's all
352 * column families.
354 default CompletableFuture<Void> compact(TableName tableName, byte[] columnFamily) {
355 return compact(tableName, columnFamily, CompactType.NORMAL);
359 * Compact a table. When the returned CompletableFuture is done, it only means the compact request
360 * was sent to HBase and may need some time to finish the compact operation.
361 * Throws {@link org.apache.hadoop.hbase.TableNotFoundException} if table not found for
362 * normal compaction type.
363 * @param tableName table to compact
364 * @param compactType {@link org.apache.hadoop.hbase.client.CompactType}
366 CompletableFuture<Void> compact(TableName tableName, CompactType compactType);
369 * Compact a column family within a table. When the returned CompletableFuture is done, it only
370 * means the compact request was sent to HBase and may need some time to finish the compact
371 * operation.
372 * Throws {@link org.apache.hadoop.hbase.TableNotFoundException} if table not found for
373 * normal compaction type.
374 * @param tableName table to compact
375 * @param columnFamily column family within a table
376 * @param compactType {@link org.apache.hadoop.hbase.client.CompactType}
378 CompletableFuture<Void> compact(TableName tableName, byte[] columnFamily,
379 CompactType compactType);
382 * Compact an individual region. When the returned CompletableFuture is done, it only means the
383 * compact request was sent to HBase and may need some time to finish the compact operation.
384 * @param regionName region to compact
386 CompletableFuture<Void> compactRegion(byte[] regionName);
389 * Compact a column family within a region. When the returned CompletableFuture is done, it only
390 * means the compact request was sent to HBase and may need some time to finish the compact
391 * operation.
392 * @param regionName region to compact
393 * @param columnFamily column family within a region. If not present, compact the region's all
394 * column families.
396 CompletableFuture<Void> compactRegion(byte[] regionName, byte[] columnFamily);
399 * Major compact a table. When the returned CompletableFuture is done, it only means the compact
400 * request was sent to HBase and may need some time to finish the compact operation.
401 * Throws {@link org.apache.hadoop.hbase.TableNotFoundException} if table not found.
402 * @param tableName table to major compact
404 default CompletableFuture<Void> majorCompact(TableName tableName) {
405 return majorCompact(tableName, CompactType.NORMAL);
409 * Major compact a column family within a table. When the returned CompletableFuture is done, it
410 * only means the compact request was sent to HBase and may need some time to finish the compact
411 * operation.
412 * Throws {@link org.apache.hadoop.hbase.TableNotFoundException} if table not found for
413 * normal compaction. type.
414 * @param tableName table to major compact
415 * @param columnFamily column family within a table. If not present, major compact the table's all
416 * column families.
418 default CompletableFuture<Void> majorCompact(TableName tableName, byte[] columnFamily) {
419 return majorCompact(tableName, columnFamily, CompactType.NORMAL);
423 * Major compact a table. When the returned CompletableFuture is done, it only means the compact
424 * request was sent to HBase and may need some time to finish the compact operation.
425 * Throws {@link org.apache.hadoop.hbase.TableNotFoundException} if table not found for
426 * normal compaction type.
427 * @param tableName table to major compact
428 * @param compactType {@link org.apache.hadoop.hbase.client.CompactType}
430 CompletableFuture<Void> majorCompact(TableName tableName, CompactType compactType);
433 * Major compact a column family within a table. When the returned CompletableFuture is done, it
434 * only means the compact request was sent to HBase and may need some time to finish the compact
435 * operation.
436 * Throws {@link org.apache.hadoop.hbase.TableNotFoundException} if table not found.
437 * @param tableName table to major compact
438 * @param columnFamily column family within a table. If not present, major compact the table's all
439 * column families.
440 * @param compactType {@link org.apache.hadoop.hbase.client.CompactType}
442 CompletableFuture<Void> majorCompact(TableName tableName, byte[] columnFamily,
443 CompactType compactType);
446 * Major compact a region. When the returned CompletableFuture is done, it only means the compact
447 * request was sent to HBase and may need some time to finish the compact operation.
448 * @param regionName region to major compact
450 CompletableFuture<Void> majorCompactRegion(byte[] regionName);
453 * Major compact a column family within region. When the returned CompletableFuture is done, it
454 * only means the compact request was sent to HBase and may need some time to finish the compact
455 * operation.
456 * @param regionName region to major compact
457 * @param columnFamily column family within a region. If not present, major compact the region's
458 * all column families.
460 CompletableFuture<Void> majorCompactRegion(byte[] regionName, byte[] columnFamily);
463 * Compact all regions on the region server.
464 * @param serverName the region server name
466 CompletableFuture<Void> compactRegionServer(ServerName serverName);
469 * Compact all regions on the region server.
470 * @param serverName the region server name
472 CompletableFuture<Void> majorCompactRegionServer(ServerName serverName);
475 * Turn the Merge switch on or off.
476 * @param enabled enabled or not
477 * @return Previous switch value wrapped by a {@link CompletableFuture}
479 default CompletableFuture<Boolean> mergeSwitch(boolean enabled) {
480 return mergeSwitch(enabled, false);
484 * Turn the Merge switch on or off.
485 * <p/>
486 * Notice that, the method itself is always non-blocking, which means it will always return
487 * immediately. The {@code drainMerges} parameter only effects when will we complete the returned
488 * {@link CompletableFuture}.
489 * @param enabled enabled or not
490 * @param drainMerges If <code>true</code>, it waits until current merge() call, if outstanding,
491 * to return.
492 * @return Previous switch value wrapped by a {@link CompletableFuture}
494 CompletableFuture<Boolean> mergeSwitch(boolean enabled, boolean drainMerges);
497 * Query the current state of the Merge switch.
498 * @return true if the switch is on, false otherwise. The return value will be wrapped by a
499 * {@link CompletableFuture}
501 CompletableFuture<Boolean> isMergeEnabled();
504 * Turn the Split switch on or off.
505 * @param enabled enabled or not
506 * @return Previous switch value wrapped by a {@link CompletableFuture}
508 default CompletableFuture<Boolean> splitSwitch(boolean enabled) {
509 return splitSwitch(enabled, false);
513 * Turn the Split switch on or off.
514 * <p/>
515 * Notice that, the method itself is always non-blocking, which means it will always return
516 * immediately. The {@code drainSplits} parameter only effects when will we complete the returned
517 * {@link CompletableFuture}.
518 * @param enabled enabled or not
519 * @param drainSplits If <code>true</code>, it waits until current split() call, if outstanding,
520 * to return.
521 * @return Previous switch value wrapped by a {@link CompletableFuture}
523 CompletableFuture<Boolean> splitSwitch(boolean enabled, boolean drainSplits);
526 * Query the current state of the Split switch.
527 * @return true if the switch is on, false otherwise. The return value will be wrapped by a
528 * {@link CompletableFuture}
530 CompletableFuture<Boolean> isSplitEnabled();
533 * Merge two regions.
534 * @param nameOfRegionA encoded or full name of region a
535 * @param nameOfRegionB encoded or full name of region b
536 * @param forcible true if do a compulsory merge, otherwise we will only merge two adjacent
537 * regions
538 * @deprecated since 2.3.0 and will be removed in 4.0.0.Use {@link #mergeRegions(List, boolean)}
539 * instead.
541 @Deprecated
542 default CompletableFuture<Void> mergeRegions(byte[] nameOfRegionA, byte[] nameOfRegionB,
543 boolean forcible) {
544 return mergeRegions(Arrays.asList(nameOfRegionA, nameOfRegionB), forcible);
548 * Merge multiple regions (>=2).
549 * @param nameOfRegionsToMerge encoded or full name of daughter regions
550 * @param forcible true if do a compulsory merge, otherwise we will only merge two adjacent
551 * regions
553 CompletableFuture<Void> mergeRegions(List<byte[]> nameOfRegionsToMerge, boolean forcible);
556 * Split a table. The method will execute split action for each region in table.
557 * @param tableName table to split
559 CompletableFuture<Void> split(TableName tableName);
562 * Split an individual region.
563 * @param regionName region to split
565 CompletableFuture<Void> splitRegion(byte[] regionName);
568 * Split a table.
569 * @param tableName table to split
570 * @param splitPoint the explicit position to split on
572 CompletableFuture<Void> split(TableName tableName, byte[] splitPoint);
575 * Split an individual region.
576 * @param regionName region to split
577 * @param splitPoint the explicit position to split on. If not present, it will decide by region
578 * server.
580 CompletableFuture<Void> splitRegion(byte[] regionName, byte[] splitPoint);
583 * @param regionName Encoded or full name of region to assign.
585 CompletableFuture<Void> assign(byte[] regionName);
588 * @param regionName Encoded or full name of region to unassign.
590 CompletableFuture<Void> unassign(byte[] regionName);
593 * Unassign a region from current hosting regionserver. Region will then be assigned to a
594 * regionserver chosen at random. Region could be reassigned back to the same server. Use
595 * {@link #move(byte[], ServerName)} if you want to control the region movement.
596 * @param regionName Encoded or full name of region to unassign. Will clear any existing
597 * RegionPlan if one found.
598 * @param forcible If true, force unassign (Will remove region from regions-in-transition too if
599 * present. If results in double assignment use hbck -fix to resolve. To be used by
600 * experts).
601 * @deprecated since 2.4.0 and will be removed in 4.0.0. Use {@link #unassign(byte[])}
602 * instead.
603 * @see <a href="https://issues.apache.org/jira/browse/HBASE-24875">HBASE-24875</a>
605 @Deprecated
606 default CompletableFuture<Void> unassign(byte[] regionName, boolean forcible) {
607 return unassign(regionName);
611 * Offline specified region from master's in-memory state. It will not attempt to reassign the
612 * region as in unassign. This API can be used when a region not served by any region server and
613 * still online as per Master's in memory state. If this API is incorrectly used on active region
614 * then master will loose track of that region. This is a special method that should be used by
615 * experts or hbck.
616 * @param regionName Encoded or full name of region to offline
618 CompletableFuture<Void> offline(byte[] regionName);
621 * Move the region <code>r</code> to a random server.
622 * @param regionName Encoded or full name of region to move.
624 CompletableFuture<Void> move(byte[] regionName);
627 * Move the region <code>r</code> to <code>dest</code>.
628 * @param regionName Encoded or full name of region to move.
629 * @param destServerName The servername of the destination regionserver. If not present, we'll
630 * assign to a random server. A server name is made of host, port and startcode. Here is
631 * an example: <code> host187.example.com,60020,1289493121758</code>
633 CompletableFuture<Void> move(byte[] regionName, ServerName destServerName);
636 * Apply the new quota settings.
637 * @param quota the quota settings
639 CompletableFuture<Void> setQuota(QuotaSettings quota);
642 * List the quotas based on the filter.
643 * @param filter the quota settings filter
644 * @return the QuotaSetting list, which wrapped by a CompletableFuture.
646 CompletableFuture<List<QuotaSettings>> getQuota(QuotaFilter filter);
649 * Add a new replication peer for replicating data to slave cluster
650 * @param peerId a short name that identifies the peer
651 * @param peerConfig configuration for the replication slave cluster
653 default CompletableFuture<Void> addReplicationPeer(String peerId,
654 ReplicationPeerConfig peerConfig) {
655 return addReplicationPeer(peerId, peerConfig, true);
659 * Add a new replication peer for replicating data to slave cluster
660 * @param peerId a short name that identifies the peer
661 * @param peerConfig configuration for the replication slave cluster
662 * @param enabled peer state, true if ENABLED and false if DISABLED
664 CompletableFuture<Void> addReplicationPeer(String peerId,
665 ReplicationPeerConfig peerConfig, boolean enabled);
668 * Remove a peer and stop the replication
669 * @param peerId a short name that identifies the peer
671 CompletableFuture<Void> removeReplicationPeer(String peerId);
674 * Restart the replication stream to the specified peer
675 * @param peerId a short name that identifies the peer
677 CompletableFuture<Void> enableReplicationPeer(String peerId);
680 * Stop the replication stream to the specified peer
681 * @param peerId a short name that identifies the peer
683 CompletableFuture<Void> disableReplicationPeer(String peerId);
686 * Returns the configured ReplicationPeerConfig for the specified peer
687 * @param peerId a short name that identifies the peer
688 * @return ReplicationPeerConfig for the peer wrapped by a {@link CompletableFuture}.
690 CompletableFuture<ReplicationPeerConfig> getReplicationPeerConfig(String peerId);
693 * Update the peerConfig for the specified peer
694 * @param peerId a short name that identifies the peer
695 * @param peerConfig new config for the peer
697 CompletableFuture<Void> updateReplicationPeerConfig(String peerId,
698 ReplicationPeerConfig peerConfig);
701 * Transit current cluster to a new state in a synchronous replication peer.
702 * @param peerId a short name that identifies the peer
703 * @param state a new state of current cluster
705 CompletableFuture<Void> transitReplicationPeerSyncReplicationState(String peerId,
706 SyncReplicationState state);
709 * Get the current cluster state in a synchronous replication peer.
710 * @param peerId a short name that identifies the peer
711 * @return the current cluster state wrapped by a {@link CompletableFuture}.
713 default CompletableFuture<SyncReplicationState> getReplicationPeerSyncReplicationState(
714 String peerId) {
715 CompletableFuture<SyncReplicationState> future = new CompletableFuture<>();
716 addListener(listReplicationPeers(Pattern.compile(peerId)), (peers, error) -> {
717 if (error != null) {
718 future.completeExceptionally(error);
719 } else if (peers.isEmpty() || !peers.get(0).getPeerId().equals(peerId)) {
720 future
721 .completeExceptionally(new IOException("Replication peer " + peerId + " does not exist"));
722 } else {
723 future.complete(peers.get(0).getSyncReplicationState());
726 return future;
730 * Append the replicable table-cf config of the specified peer
731 * @param peerId a short that identifies the cluster
732 * @param tableCfs A map from tableName to column family names
734 CompletableFuture<Void> appendReplicationPeerTableCFs(String peerId,
735 Map<TableName, List<String>> tableCfs);
738 * Remove some table-cfs from config of the specified peer
739 * @param peerId a short name that identifies the cluster
740 * @param tableCfs A map from tableName to column family names
742 CompletableFuture<Void> removeReplicationPeerTableCFs(String peerId,
743 Map<TableName, List<String>> tableCfs);
746 * Return a list of replication peers.
747 * @return a list of replication peers description. The return value will be wrapped by a
748 * {@link CompletableFuture}.
750 CompletableFuture<List<ReplicationPeerDescription>> listReplicationPeers();
753 * Return a list of replication peers.
754 * @param pattern The compiled regular expression to match peer id
755 * @return a list of replication peers description. The return value will be wrapped by a
756 * {@link CompletableFuture}.
758 CompletableFuture<List<ReplicationPeerDescription>> listReplicationPeers(Pattern pattern);
761 * Find all table and column families that are replicated from this cluster
762 * @return the replicated table-cfs list of this cluster. The return value will be wrapped by a
763 * {@link CompletableFuture}.
765 CompletableFuture<List<TableCFs>> listReplicatedTableCFs();
768 * Enable a table's replication switch.
769 * @param tableName name of the table
771 CompletableFuture<Void> enableTableReplication(TableName tableName);
774 * Disable a table's replication switch.
775 * @param tableName name of the table
777 CompletableFuture<Void> disableTableReplication(TableName tableName);
780 * Take a snapshot for the given table. If the table is enabled, a FLUSH-type snapshot will be
781 * taken. If the table is disabled, an offline snapshot is taken. Snapshots are taken
782 * sequentially even when requested concurrently, across all tables. Snapshots are considered
783 * unique based on <b>the name of the snapshot</b>. Attempts to take a snapshot with the same
784 * name (even a different type or with different parameters) will fail with a
785 * {@link org.apache.hadoop.hbase.snapshot.SnapshotCreationException} indicating the duplicate
786 * naming. Snapshot names follow the same naming constraints as tables in HBase. See
787 * {@link org.apache.hadoop.hbase.TableName#isLegalFullyQualifiedTableName(byte[])}.
788 * @param snapshotName name of the snapshot to be created
789 * @param tableName name of the table for which snapshot is created
791 default CompletableFuture<Void> snapshot(String snapshotName, TableName tableName) {
792 return snapshot(snapshotName, tableName, SnapshotType.FLUSH);
796 * Create typed snapshot of the table. Snapshots are considered unique based on <b>the name of the
797 * snapshot</b>. Snapshots are taken sequentially even when requested concurrently, across all
798 * tables. Attempts to take a snapshot with the same name (even a different type or with
799 * different parameters) will fail with a
800 * {@link org.apache.hadoop.hbase.snapshot.SnapshotCreationException} indicating the duplicate
801 * naming. Snapshot names follow the same naming constraints as tables in HBase. See
802 * {@link org.apache.hadoop.hbase.TableName#isLegalFullyQualifiedTableName(byte[])}.
803 * @param snapshotName name to give the snapshot on the filesystem. Must be unique from all other
804 * snapshots stored on the cluster
805 * @param tableName name of the table to snapshot
806 * @param type type of snapshot to take
808 default CompletableFuture<Void> snapshot(String snapshotName, TableName tableName,
809 SnapshotType type) {
810 return snapshot(new SnapshotDescription(snapshotName, tableName, type));
814 * Take a snapshot and wait for the server to complete that snapshot asynchronously. Snapshots
815 * are taken sequentially even when requested concurrently, across all tables. Snapshots are
816 * considered unique based on <b>the name of the snapshot</b>.
817 * Attempts to take a snapshot with the same name (even a different type or with different
818 * parameters) will fail with a {@link org.apache.hadoop.hbase.snapshot.SnapshotCreationException}
819 * indicating the duplicate naming. Snapshot names follow the same naming constraints as tables in
820 * HBase. See {@link org.apache.hadoop.hbase.TableName#isLegalFullyQualifiedTableName(byte[])}.
821 * You should probably use {@link #snapshot(String, org.apache.hadoop.hbase.TableName)} unless you
822 * are sure about the type of snapshot that you want to take.
823 * @param snapshot snapshot to take
825 CompletableFuture<Void> snapshot(SnapshotDescription snapshot);
828 * Check the current state of the passed snapshot. There are three possible states:
829 * <ol>
830 * <li>running - returns <tt>false</tt></li>
831 * <li>finished - returns <tt>true</tt></li>
832 * <li>finished with error - throws the exception that caused the snapshot to fail</li>
833 * </ol>
834 * The cluster only knows about the most recent snapshot. Therefore, if another snapshot has been
835 * run/started since the snapshot you are checking, you will receive an
836 * {@link org.apache.hadoop.hbase.snapshot.UnknownSnapshotException}.
837 * @param snapshot description of the snapshot to check
838 * @return <tt>true</tt> if the snapshot is completed, <tt>false</tt> if the snapshot is still
839 * running
841 CompletableFuture<Boolean> isSnapshotFinished(SnapshotDescription snapshot);
844 * Restore the specified snapshot on the original table. (The table must be disabled) If the
845 * "hbase.snapshot.restore.take.failsafe.snapshot" configuration property is set to true, a
846 * snapshot of the current table is taken before executing the restore operation. In case of
847 * restore failure, the failsafe snapshot will be restored. If the restore completes without
848 * problem the failsafe snapshot is deleted.
849 * @param snapshotName name of the snapshot to restore
851 CompletableFuture<Void> restoreSnapshot(String snapshotName);
854 * Restore the specified snapshot on the original table. (The table must be disabled) If
855 * 'takeFailSafeSnapshot' is set to true, a snapshot of the current table is taken before
856 * executing the restore operation. In case of restore failure, the failsafe snapshot will be
857 * restored. If the restore completes without problem the failsafe snapshot is deleted. The
858 * failsafe snapshot name is configurable by using the property
859 * "hbase.snapshot.restore.failsafe.name".
860 * @param snapshotName name of the snapshot to restore
861 * @param takeFailSafeSnapshot true if the failsafe snapshot should be taken
863 default CompletableFuture<Void> restoreSnapshot(String snapshotName,
864 boolean takeFailSafeSnapshot) {
865 return restoreSnapshot(snapshotName, takeFailSafeSnapshot, false);
869 * Restore the specified snapshot on the original table. (The table must be disabled) If
870 * 'takeFailSafeSnapshot' is set to true, a snapshot of the current table is taken before
871 * executing the restore operation. In case of restore failure, the failsafe snapshot will be
872 * restored. If the restore completes without problem the failsafe snapshot is deleted. The
873 * failsafe snapshot name is configurable by using the property
874 * "hbase.snapshot.restore.failsafe.name".
875 * @param snapshotName name of the snapshot to restore
876 * @param takeFailSafeSnapshot true if the failsafe snapshot should be taken
877 * @param restoreAcl <code>true</code> to restore acl of snapshot
879 CompletableFuture<Void> restoreSnapshot(String snapshotName, boolean takeFailSafeSnapshot,
880 boolean restoreAcl);
883 * Create a new table by cloning the snapshot content.
884 * @param snapshotName name of the snapshot to be cloned
885 * @param tableName name of the table where the snapshot will be restored
887 default CompletableFuture<Void> cloneSnapshot(String snapshotName, TableName tableName) {
888 return cloneSnapshot(snapshotName, tableName, false);
892 * Create a new table by cloning the snapshot content.
893 * @param snapshotName name of the snapshot to be cloned
894 * @param tableName name of the table where the snapshot will be restored
895 * @param restoreAcl <code>true</code> to restore acl of snapshot
897 CompletableFuture<Void> cloneSnapshot(String snapshotName, TableName tableName,
898 boolean restoreAcl);
901 * List completed snapshots.
902 * @return a list of snapshot descriptors for completed snapshots wrapped by a
903 * {@link CompletableFuture}
905 CompletableFuture<List<SnapshotDescription>> listSnapshots();
908 * List all the completed snapshots matching the given pattern.
909 * @param pattern The compiled regular expression to match against
910 * @return - returns a List of SnapshotDescription wrapped by a {@link CompletableFuture}
912 CompletableFuture<List<SnapshotDescription>> listSnapshots(Pattern pattern);
915 * List all the completed snapshots matching the given table name pattern.
916 * @param tableNamePattern The compiled table name regular expression to match against
917 * @return - returns a List of completed SnapshotDescription wrapped by a
918 * {@link CompletableFuture}
920 CompletableFuture<List<SnapshotDescription>> listTableSnapshots(Pattern tableNamePattern);
923 * List all the completed snapshots matching the given table name regular expression and snapshot
924 * name regular expression.
925 * @param tableNamePattern The compiled table name regular expression to match against
926 * @param snapshotNamePattern The compiled snapshot name regular expression to match against
927 * @return - returns a List of completed SnapshotDescription wrapped by a
928 * {@link CompletableFuture}
930 CompletableFuture<List<SnapshotDescription>> listTableSnapshots(Pattern tableNamePattern,
931 Pattern snapshotNamePattern);
934 * Delete an existing snapshot.
935 * @param snapshotName name of the snapshot
937 CompletableFuture<Void> deleteSnapshot(String snapshotName);
940 * Delete all existing snapshots.
942 CompletableFuture<Void> deleteSnapshots();
945 * Delete existing snapshots whose names match the pattern passed.
946 * @param pattern pattern for names of the snapshot to match
948 CompletableFuture<Void> deleteSnapshots(Pattern pattern);
951 * Delete all existing snapshots matching the given table name pattern.
952 * @param tableNamePattern The compiled table name regular expression to match against
954 CompletableFuture<Void> deleteTableSnapshots(Pattern tableNamePattern);
957 * Delete all existing snapshots matching the given table name regular expression and snapshot
958 * name regular expression.
959 * @param tableNamePattern The compiled table name regular expression to match against
960 * @param snapshotNamePattern The compiled snapshot name regular expression to match against
962 CompletableFuture<Void> deleteTableSnapshots(Pattern tableNamePattern,
963 Pattern snapshotNamePattern);
966 * Execute a distributed procedure on a cluster.
967 * @param signature A distributed procedure is uniquely identified by its signature (default the
968 * root ZK node name of the procedure).
969 * @param instance The instance name of the procedure. For some procedures, this parameter is
970 * optional.
971 * @param props Property/Value pairs of properties passing to the procedure
973 CompletableFuture<Void> execProcedure(String signature, String instance,
974 Map<String, String> props);
977 * Execute a distributed procedure on a cluster.
978 * @param signature A distributed procedure is uniquely identified by its signature (default the
979 * root ZK node name of the procedure).
980 * @param instance The instance name of the procedure. For some procedures, this parameter is
981 * optional.
982 * @param props Property/Value pairs of properties passing to the procedure
983 * @return data returned after procedure execution. null if no return data.
985 CompletableFuture<byte[]> execProcedureWithReturn(String signature, String instance,
986 Map<String, String> props);
989 * Check the current state of the specified procedure. There are three possible states:
990 * <ol>
991 * <li>running - returns <tt>false</tt></li>
992 * <li>finished - returns <tt>true</tt></li>
993 * <li>finished with error - throws the exception that caused the procedure to fail</li>
994 * </ol>
995 * @param signature The signature that uniquely identifies a procedure
996 * @param instance The instance name of the procedure
997 * @param props Property/Value pairs of properties passing to the procedure
998 * @return true if the specified procedure is finished successfully, false if it is still running.
999 * The value is wrapped by {@link CompletableFuture}
1001 CompletableFuture<Boolean> isProcedureFinished(String signature, String instance,
1002 Map<String, String> props);
1005 * Abort a procedure
1006 * Do not use. Usually it is ignored but if not, it can do more damage than good. See hbck2.
1007 * @param procId ID of the procedure to abort
1008 * @param mayInterruptIfRunning if the proc completed at least one step, should it be aborted?
1009 * @return true if aborted, false if procedure already completed or does not exist. the value is
1010 * wrapped by {@link CompletableFuture}
1011 * @deprecated since 2.1.1 and will be removed in 4.0.0.
1012 * @see <a href="https://issues.apache.org/jira/browse/HBASE-21223">HBASE-21223</a>
1014 @Deprecated
1015 CompletableFuture<Boolean> abortProcedure(long procId, boolean mayInterruptIfRunning);
1018 * List procedures
1019 * @return procedure list JSON wrapped by {@link CompletableFuture}
1021 CompletableFuture<String> getProcedures();
1024 * List locks.
1025 * @return lock list JSON wrapped by {@link CompletableFuture}
1027 CompletableFuture<String> getLocks();
1030 * Mark region server(s) as decommissioned to prevent additional regions from getting
1031 * assigned to them. Optionally unload the regions on the servers. If there are multiple servers
1032 * to be decommissioned, decommissioning them at the same time can prevent wasteful region
1033 * movements. Region unloading is asynchronous.
1034 * @param servers The list of servers to decommission.
1035 * @param offload True to offload the regions from the decommissioned servers
1037 CompletableFuture<Void> decommissionRegionServers(List<ServerName> servers, boolean offload);
1040 * List region servers marked as decommissioned, which can not be assigned regions.
1041 * @return List of decommissioned region servers wrapped by {@link CompletableFuture}
1043 CompletableFuture<List<ServerName>> listDecommissionedRegionServers();
1046 * Remove decommission marker from a region server to allow regions assignments. Load regions onto
1047 * the server if a list of regions is given. Region loading is asynchronous.
1048 * @param server The server to recommission.
1049 * @param encodedRegionNames Regions to load onto the server.
1051 CompletableFuture<Void> recommissionRegionServer(ServerName server,
1052 List<byte[]> encodedRegionNames);
1055 * @return cluster status wrapped by {@link CompletableFuture}
1057 CompletableFuture<ClusterMetrics> getClusterMetrics();
1060 * @return cluster status wrapped by {@link CompletableFuture}
1062 CompletableFuture<ClusterMetrics> getClusterMetrics(EnumSet<Option> options);
1065 * @return current master server name wrapped by {@link CompletableFuture}
1067 default CompletableFuture<ServerName> getMaster() {
1068 return getClusterMetrics(EnumSet.of(Option.MASTER)).thenApply(ClusterMetrics::getMasterName);
1072 * @return current backup master list wrapped by {@link CompletableFuture}
1074 default CompletableFuture<Collection<ServerName>> getBackupMasters() {
1075 return getClusterMetrics(EnumSet.of(Option.BACKUP_MASTERS))
1076 .thenApply(ClusterMetrics::getBackupMasterNames);
1080 * @return current live region servers list wrapped by {@link CompletableFuture}
1082 default CompletableFuture<Collection<ServerName>> getRegionServers() {
1083 return getClusterMetrics(EnumSet.of(Option.SERVERS_NAME))
1084 .thenApply(ClusterMetrics::getServersName);
1088 * @return a list of master coprocessors wrapped by {@link CompletableFuture}
1090 default CompletableFuture<List<String>> getMasterCoprocessorNames() {
1091 return getClusterMetrics(EnumSet.of(Option.MASTER_COPROCESSORS))
1092 .thenApply(ClusterMetrics::getMasterCoprocessorNames);
1096 * Get the info port of the current master if one is available.
1097 * @return master info port
1099 default CompletableFuture<Integer> getMasterInfoPort() {
1100 return getClusterMetrics(EnumSet.of(Option.MASTER_INFO_PORT)).thenApply(
1101 ClusterMetrics::getMasterInfoPort);
1105 * Shuts down the HBase cluster.
1107 CompletableFuture<Void> shutdown();
1110 * Shuts down the current HBase master only.
1112 CompletableFuture<Void> stopMaster();
1115 * Stop the designated regionserver.
1116 * @param serverName
1118 CompletableFuture<Void> stopRegionServer(ServerName serverName);
1121 * Update the configuration and trigger an online config change on the regionserver.
1122 * @param serverName : The server whose config needs to be updated.
1124 CompletableFuture<Void> updateConfiguration(ServerName serverName);
1127 * Update the configuration and trigger an online config change on all the masters and
1128 * regionservers.
1130 CompletableFuture<Void> updateConfiguration();
1133 * Roll the log writer. I.e. for filesystem based write ahead logs, start writing to a new file.
1134 * <p>
1135 * When the returned CompletableFuture is done, it only means the rollWALWriter request was sent
1136 * to the region server and may need some time to finish the rollWALWriter operation. As a side
1137 * effect of this call, the named region server may schedule store flushes at the request of the
1138 * wal.
1139 * @param serverName The servername of the region server.
1141 CompletableFuture<Void> rollWALWriter(ServerName serverName);
1144 * Clear compacting queues on a region server.
1145 * @param serverName
1146 * @param queues the set of queue name
1148 CompletableFuture<Void> clearCompactionQueues(ServerName serverName, Set<String> queues);
1151 * Get a list of {@link RegionMetrics} of all regions hosted on a region seerver.
1152 * @param serverName
1153 * @return a list of {@link RegionMetrics} wrapped by {@link CompletableFuture}
1155 CompletableFuture<List<RegionMetrics>> getRegionMetrics(ServerName serverName);
1158 * Get a list of {@link RegionMetrics} of all regions hosted on a region seerver for a table.
1159 * @param serverName
1160 * @param tableName
1161 * @return a list of {@link RegionMetrics} wrapped by {@link CompletableFuture}
1163 CompletableFuture<List<RegionMetrics>> getRegionMetrics(ServerName serverName,
1164 TableName tableName);
1167 * Check whether master is in maintenance mode
1168 * @return true if master is in maintenance mode, false otherwise. The return value will be
1169 * wrapped by a {@link CompletableFuture}
1171 CompletableFuture<Boolean> isMasterInMaintenanceMode();
1174 * Get the current compaction state of a table. It could be in a major compaction, a minor
1175 * compaction, both, or none.
1176 * @param tableName table to examine
1177 * @return the current compaction state wrapped by a {@link CompletableFuture}
1179 default CompletableFuture<CompactionState> getCompactionState(TableName tableName) {
1180 return getCompactionState(tableName, CompactType.NORMAL);
1184 * Get the current compaction state of a table. It could be in a major compaction, a minor
1185 * compaction, both, or none.
1186 * @param tableName table to examine
1187 * @param compactType {@link org.apache.hadoop.hbase.client.CompactType}
1188 * @return the current compaction state wrapped by a {@link CompletableFuture}
1190 CompletableFuture<CompactionState> getCompactionState(TableName tableName,
1191 CompactType compactType);
1194 * Get the current compaction state of region. It could be in a major compaction, a minor
1195 * compaction, both, or none.
1196 * @param regionName region to examine
1197 * @return the current compaction state wrapped by a {@link CompletableFuture}
1199 CompletableFuture<CompactionState> getCompactionStateForRegion(byte[] regionName);
1202 * Get the timestamp of the last major compaction for the passed table.
1203 * <p>
1204 * The timestamp of the oldest HFile resulting from a major compaction of that table, or not
1205 * present if no such HFile could be found.
1206 * @param tableName table to examine
1207 * @return the last major compaction timestamp wrapped by a {@link CompletableFuture}
1209 CompletableFuture<Optional<Long>> getLastMajorCompactionTimestamp(TableName tableName);
1212 * Get the timestamp of the last major compaction for the passed region.
1213 * <p>
1214 * The timestamp of the oldest HFile resulting from a major compaction of that region, or not
1215 * present if no such HFile could be found.
1216 * @param regionName region to examine
1217 * @return the last major compaction timestamp wrapped by a {@link CompletableFuture}
1219 CompletableFuture<Optional<Long>> getLastMajorCompactionTimestampForRegion(byte[] regionName);
1222 * @return the list of supported security capabilities. The return value will be wrapped by a
1223 * {@link CompletableFuture}.
1225 CompletableFuture<List<SecurityCapability>> getSecurityCapabilities();
1228 * Turn the load balancer on or off.
1229 * @param on Set to <code>true</code> to enable, <code>false</code> to disable.
1230 * @return Previous balancer value wrapped by a {@link CompletableFuture}.
1232 default CompletableFuture<Boolean> balancerSwitch(boolean on) {
1233 return balancerSwitch(on, false);
1237 * Turn the load balancer on or off.
1238 * <p/>
1239 * Notice that, the method itself is always non-blocking, which means it will always return
1240 * immediately. The {@code drainRITs} parameter only effects when will we complete the returned
1241 * {@link CompletableFuture}.
1242 * @param on Set to <code>true</code> to enable, <code>false</code> to disable.
1243 * @param drainRITs If <code>true</code>, it waits until current balance() call, if outstanding,
1244 * to return.
1245 * @return Previous balancer value wrapped by a {@link CompletableFuture}.
1247 CompletableFuture<Boolean> balancerSwitch(boolean on, boolean drainRITs);
1250 * Invoke the balancer. Will run the balancer and if regions to move, it will go ahead and do the
1251 * reassignments. Can NOT run for various reasons. Check logs.
1252 * @return True if balancer ran, false otherwise. The return value will be wrapped by a
1253 * {@link CompletableFuture}.
1255 default CompletableFuture<Boolean> balance() {
1256 return balance(false);
1260 * Invoke the balancer. Will run the balancer and if regions to move, it will go ahead and do the
1261 * reassignments. If there is region in transition, force parameter of true would still run
1262 * balancer. Can *not* run for other reasons. Check logs.
1263 * @param forcible whether we should force balance even if there is region in transition.
1264 * @return True if balancer ran, false otherwise. The return value will be wrapped by a
1265 * {@link CompletableFuture}.
1267 CompletableFuture<Boolean> balance(boolean forcible);
1270 * Query the current state of the balancer.
1271 * @return true if the balance switch is on, false otherwise. The return value will be wrapped by a
1272 * {@link CompletableFuture}.
1274 CompletableFuture<Boolean> isBalancerEnabled();
1277 * Set region normalizer on/off.
1278 * @param on whether normalizer should be on or off
1279 * @return Previous normalizer value wrapped by a {@link CompletableFuture}
1281 CompletableFuture<Boolean> normalizerSwitch(boolean on);
1284 * Query the current state of the region normalizer
1285 * @return true if region normalizer is on, false otherwise. The return value will be wrapped by a
1286 * {@link CompletableFuture}
1288 CompletableFuture<Boolean> isNormalizerEnabled();
1291 * Invoke region normalizer. Can NOT run for various reasons. Check logs.
1292 * @return true if region normalizer ran, false otherwise. The return value will be wrapped by a
1293 * {@link CompletableFuture}
1295 default CompletableFuture<Boolean> normalize() {
1296 return normalize(new NormalizeTableFilterParams.Builder().build());
1300 * Invoke region normalizer. Can NOT run for various reasons. Check logs.
1301 * @param ntfp limit to tables matching the specified filter.
1302 * @return true if region normalizer ran, false otherwise. The return value will be wrapped by a
1303 * {@link CompletableFuture}
1305 CompletableFuture<Boolean> normalize(NormalizeTableFilterParams ntfp);
1308 * Turn the cleaner chore on/off.
1309 * @param on
1310 * @return Previous cleaner state wrapped by a {@link CompletableFuture}
1312 CompletableFuture<Boolean> cleanerChoreSwitch(boolean on);
1315 * Query the current state of the cleaner chore.
1316 * @return true if cleaner chore is on, false otherwise. The return value will be wrapped by
1317 * a {@link CompletableFuture}
1319 CompletableFuture<Boolean> isCleanerChoreEnabled();
1322 * Ask for cleaner chore to run.
1323 * @return true if cleaner chore ran, false otherwise. The return value will be wrapped by a
1324 * {@link CompletableFuture}
1326 CompletableFuture<Boolean> runCleanerChore();
1329 * Turn the catalog janitor on/off.
1330 * @param on
1331 * @return the previous state wrapped by a {@link CompletableFuture}
1333 CompletableFuture<Boolean> catalogJanitorSwitch(boolean on);
1336 * Query on the catalog janitor state.
1337 * @return true if the catalog janitor is on, false otherwise. The return value will be
1338 * wrapped by a {@link CompletableFuture}
1340 CompletableFuture<Boolean> isCatalogJanitorEnabled();
1343 * Ask for a scan of the catalog table.
1344 * @return the number of entries cleaned. The return value will be wrapped by a
1345 * {@link CompletableFuture}
1347 CompletableFuture<Integer> runCatalogJanitor();
1350 * Execute the given coprocessor call on the master.
1351 * <p>
1352 * The {@code stubMaker} is just a delegation to the {@code newStub} call. Usually it is only a
1353 * one line lambda expression, like:
1355 * <pre>
1356 * <code>
1357 * channel -> xxxService.newStub(channel)
1358 * </code>
1359 * </pre>
1360 * @param stubMaker a delegation to the actual {@code newStub} call.
1361 * @param callable a delegation to the actual protobuf rpc call. See the comment of
1362 * {@link ServiceCaller} for more details.
1363 * @param <S> the type of the asynchronous stub
1364 * @param <R> the type of the return value
1365 * @return the return value of the protobuf rpc call, wrapped by a {@link CompletableFuture}.
1366 * @see ServiceCaller
1368 <S, R> CompletableFuture<R> coprocessorService(Function<RpcChannel, S> stubMaker,
1369 ServiceCaller<S, R> callable);
1372 * Execute the given coprocessor call on the given region server.
1373 * <p>
1374 * The {@code stubMaker} is just a delegation to the {@code newStub} call. Usually it is only a
1375 * one line lambda expression, like:
1377 * <pre>
1378 * <code>
1379 * channel -> xxxService.newStub(channel)
1380 * </code>
1381 * </pre>
1382 * @param stubMaker a delegation to the actual {@code newStub} call.
1383 * @param callable a delegation to the actual protobuf rpc call. See the comment of
1384 * {@link ServiceCaller} for more details.
1385 * @param serverName the given region server
1386 * @param <S> the type of the asynchronous stub
1387 * @param <R> the type of the return value
1388 * @return the return value of the protobuf rpc call, wrapped by a {@link CompletableFuture}.
1389 * @see ServiceCaller
1391 <S, R> CompletableFuture<R> coprocessorService(Function<RpcChannel, S> stubMaker,
1392 ServiceCaller<S, R> callable, ServerName serverName);
1395 * List all the dead region servers.
1397 default CompletableFuture<List<ServerName>> listDeadServers() {
1398 return this.getClusterMetrics(EnumSet.of(Option.DEAD_SERVERS))
1399 .thenApply(ClusterMetrics::getDeadServerNames);
1403 * Clear dead region servers from master.
1404 * @param servers list of dead region servers.
1405 * @return - returns a list of servers that not cleared wrapped by a {@link CompletableFuture}.
1407 CompletableFuture<List<ServerName>> clearDeadServers(final List<ServerName> servers);
1410 * Clear all the blocks corresponding to this table from BlockCache. For expert-admins. Calling
1411 * this API will drop all the cached blocks specific to a table from BlockCache. This can
1412 * significantly impact the query performance as the subsequent queries will have to retrieve the
1413 * blocks from underlying filesystem.
1414 * @param tableName table to clear block cache
1415 * @return CacheEvictionStats related to the eviction wrapped by a {@link CompletableFuture}.
1417 CompletableFuture<CacheEvictionStats> clearBlockCache(final TableName tableName);
1420 * Create a new table by cloning the existent table schema.
1422 * @param tableName name of the table to be cloned
1423 * @param newTableName name of the new table where the table will be created
1424 * @param preserveSplits True if the splits should be preserved
1426 CompletableFuture<Void> cloneTableSchema(final TableName tableName,
1427 final TableName newTableName, final boolean preserveSplits);
1430 * Turn the compaction on or off. Disabling compactions will also interrupt any currently ongoing
1431 * compactions. This state is ephemeral. The setting will be lost on restart. Compaction
1432 * can also be enabled/disabled by modifying configuration hbase.regionserver.compaction.enabled
1433 * in hbase-site.xml.
1435 * @param switchState Set to <code>true</code> to enable, <code>false</code> to disable.
1436 * @param serverNamesList list of region servers.
1437 * @return Previous compaction states for region servers
1439 CompletableFuture<Map<ServerName, Boolean>> compactionSwitch(boolean switchState,
1440 List<String> serverNamesList);
1443 * Switch the rpc throttle enabled state.
1444 * @param enable Set to <code>true</code> to enable, <code>false</code> to disable.
1445 * @return Previous rpc throttle enabled value
1447 CompletableFuture<Boolean> switchRpcThrottle(boolean enable);
1450 * Get if the rpc throttle is enabled.
1451 * @return True if rpc throttle is enabled
1453 CompletableFuture<Boolean> isRpcThrottleEnabled();
1456 * Switch the exceed throttle quota. If enabled, user/table/namespace throttle quota
1457 * can be exceeded if region server has availble quota.
1458 * @param enable Set to <code>true</code> to enable, <code>false</code> to disable.
1459 * @return Previous exceed throttle enabled value
1461 CompletableFuture<Boolean> exceedThrottleQuotaSwitch(boolean enable);
1464 * Fetches the table sizes on the filesystem as tracked by the HBase Master.
1466 CompletableFuture<Map<TableName, Long>> getSpaceQuotaTableSizes();
1469 * Fetches the observed {@link SpaceQuotaSnapshotView}s observed by a RegionServer.
1471 CompletableFuture<? extends Map<TableName, ? extends SpaceQuotaSnapshotView>>
1472 getRegionServerSpaceQuotaSnapshots(ServerName serverName);
1475 * Returns the Master's view of a quota on the given {@code namespace} or null if the Master has
1476 * no quota information on that namespace.
1478 CompletableFuture<? extends SpaceQuotaSnapshotView>
1479 getCurrentSpaceQuotaSnapshot(String namespace);
1482 * Returns the Master's view of a quota on the given {@code tableName} or null if the Master has
1483 * no quota information on that table.
1485 CompletableFuture<? extends SpaceQuotaSnapshotView> getCurrentSpaceQuotaSnapshot(
1486 TableName tableName);
1489 * Grants user specific permissions
1490 * @param userPermission user name and the specific permission
1491 * @param mergeExistingPermissions If set to false, later granted permissions will override
1492 * previous granted permissions. otherwise, it'll merge with previous granted
1493 * permissions.
1495 CompletableFuture<Void> grant(UserPermission userPermission, boolean mergeExistingPermissions);
1498 * Revokes user specific permissions
1499 * @param userPermission user name and the specific permission
1501 CompletableFuture<Void> revoke(UserPermission userPermission);
1504 * Get the global/namespace/table permissions for user
1505 * @param getUserPermissionsRequest A request contains which user, global, namespace or table
1506 * permissions needed
1507 * @return The user and permission list
1509 CompletableFuture<List<UserPermission>>
1510 getUserPermissions(GetUserPermissionsRequest getUserPermissionsRequest);
1513 * Check if the user has specific permissions
1514 * @param userName the user name
1515 * @param permissions the specific permission list
1516 * @return True if user has the specific permissions
1518 CompletableFuture<List<Boolean>> hasUserPermissions(String userName,
1519 List<Permission> permissions);
1522 * Check if call user has specific permissions
1523 * @param permissions the specific permission list
1524 * @return True if user has the specific permissions
1526 default CompletableFuture<List<Boolean>> hasUserPermissions(List<Permission> permissions) {
1527 return hasUserPermissions(null, permissions);
1531 * Turn on or off the auto snapshot cleanup based on TTL.
1532 * <p/>
1533 * Notice that, the method itself is always non-blocking, which means it will always return
1534 * immediately. The {@code sync} parameter only effects when will we complete the returned
1535 * {@link CompletableFuture}.
1537 * @param on Set to <code>true</code> to enable, <code>false</code> to disable.
1538 * @param sync If <code>true</code>, it waits until current snapshot cleanup is completed,
1539 * if outstanding.
1540 * @return Previous auto snapshot cleanup value wrapped by a {@link CompletableFuture}.
1542 CompletableFuture<Boolean> snapshotCleanupSwitch(boolean on, boolean sync);
1545 * Query the current state of the auto snapshot cleanup based on TTL.
1547 * @return true if the auto snapshot cleanup is enabled, false otherwise.
1548 * The return value will be wrapped by a {@link CompletableFuture}.
1550 CompletableFuture<Boolean> isSnapshotCleanupEnabled();
1553 * Retrieves online slow RPC logs from the provided list of
1554 * RegionServers
1556 * @param serverNames Server names to get slowlog responses from
1557 * @param logQueryFilter filter to be used if provided
1558 * @return Online slowlog response list. The return value wrapped by a {@link CompletableFuture}
1559 * @deprecated since 2.4.0 and will be removed in 4.0.0.
1560 * Use {@link #getLogEntries(Set, String, ServerType, int, Map)} instead.
1562 @Deprecated
1563 default CompletableFuture<List<OnlineLogRecord>> getSlowLogResponses(
1564 final Set<ServerName> serverNames, final LogQueryFilter logQueryFilter) {
1565 String logType;
1566 if (LogQueryFilter.Type.LARGE_LOG.equals(logQueryFilter.getType())) {
1567 logType = "LARGE_LOG";
1568 } else {
1569 logType = "SLOW_LOG";
1571 Map<String, Object> filterParams = new HashMap<>();
1572 filterParams.put("regionName", logQueryFilter.getRegionName());
1573 filterParams.put("clientAddress", logQueryFilter.getClientAddress());
1574 filterParams.put("tableName", logQueryFilter.getTableName());
1575 filterParams.put("userName", logQueryFilter.getUserName());
1576 filterParams.put("filterByOperator", logQueryFilter.getFilterByOperator().toString());
1577 CompletableFuture<List<LogEntry>> logEntries =
1578 getLogEntries(serverNames, logType, ServerType.REGION_SERVER, logQueryFilter.getLimit(),
1579 filterParams);
1580 return logEntries.thenApply(
1581 logEntryList -> logEntryList.stream().map(logEntry -> (OnlineLogRecord) logEntry)
1582 .collect(Collectors.toList()));
1586 * Clears online slow RPC logs from the provided list of
1587 * RegionServers
1589 * @param serverNames Set of Server names to clean slowlog responses from
1590 * @return List of booleans representing if online slowlog response buffer is cleaned
1591 * from each RegionServer. The return value wrapped by a {@link CompletableFuture}
1593 CompletableFuture<List<Boolean>> clearSlowLogResponses(final Set<ServerName> serverNames);
1596 * Creates a new RegionServer group with the given name
1597 * @param groupName the name of the group
1598 * @throws IOException if a remote or network exception occurs
1600 CompletableFuture<Void> addRSGroup(String groupName);
1603 * Get group info for the given group name
1604 * @param groupName the group name
1605 * @return group info
1606 * @throws IOException if a remote or network exception occurs
1608 CompletableFuture<RSGroupInfo> getRSGroup(String groupName);
1611 * Get group info for the given hostPort
1612 * @param hostPort HostPort to get RSGroupInfo for
1613 * @throws IOException if a remote or network exception occurs
1615 CompletableFuture<RSGroupInfo> getRSGroup(Address hostPort);
1618 * Get group info for the given table
1619 * @param tableName table name to get RSGroupInfo for
1620 * @throws IOException if a remote or network exception occurs
1622 CompletableFuture<RSGroupInfo> getRSGroup(TableName tableName);
1625 * Lists current set of RegionServer groups
1626 * @throws IOException if a remote or network exception occurs
1628 CompletableFuture<List<RSGroupInfo>> listRSGroups();
1631 * Get all tables in this RegionServer group.
1632 * @param groupName the group name
1633 * @throws IOException if a remote or network exception occurs
1634 * @see #getConfiguredNamespacesAndTablesInRSGroup(String)
1636 CompletableFuture<List<TableName>> listTablesInRSGroup(String groupName);
1639 * Get the namespaces and tables which have this RegionServer group in descriptor.
1640 * <p/>
1641 * The difference between this method and {@link #listTablesInRSGroup(String)} is that, this
1642 * method will not include the table which is actually in this RegionServr group but without the
1643 * RegionServer group configuration in its {@link TableDescriptor}. For example, we have a group
1644 * 'A', and we make namespace 'nsA' in this group, then all the tables under this namespace will
1645 * in the group 'A', but this method will not return these tables but only the namespace 'nsA',
1646 * while the {@link #listTablesInRSGroup(String)} will return all these tables.
1647 * @param groupName the group name
1648 * @throws IOException if a remote or network exception occurs
1649 * @see #listTablesInRSGroup(String)
1651 CompletableFuture<Pair<List<String>, List<TableName>>>
1652 getConfiguredNamespacesAndTablesInRSGroup(String groupName);
1655 * Remove RegionServer group associated with the given name
1656 * @param groupName the group name
1657 * @throws IOException if a remote or network exception occurs
1659 CompletableFuture<Void> removeRSGroup(String groupName);
1662 * Remove decommissioned servers from group
1663 * 1. Sometimes we may find the server aborted due to some hardware failure and we must offline
1664 * the server for repairing. Or we need to move some servers to join other clusters.
1665 * So we need to remove these servers from the group.
1666 * 2. Dead/recovering/live servers will be disallowed.
1667 * @param servers set of servers to remove
1668 * @throws IOException if a remote or network exception occurs
1670 CompletableFuture<Void> removeServersFromRSGroup(Set<Address> servers);
1673 * Move given set of servers to the specified target RegionServer group
1674 * @param servers set of servers to move
1675 * @param groupName the group to move servers to
1676 * @throws IOException if a remote or network exception occurs
1678 CompletableFuture<Void> moveServersToRSGroup(Set<Address> servers, String groupName);
1681 * Set the RegionServer group for tables
1682 * @param tables tables to set group for
1683 * @param groupName group name for tables
1684 * @throws IOException if a remote or network exception occurs
1686 CompletableFuture<Void> setRSGroup(Set<TableName> tables, String groupName);
1689 * Balance regions in the given RegionServer group
1690 * @param groupName the group name
1691 * @return boolean Whether balance ran or not
1692 * @throws IOException if a remote or network exception occurs
1694 CompletableFuture<Boolean> balanceRSGroup(String groupName);
1697 * Rename rsgroup
1698 * @param oldName old rsgroup name
1699 * @param newName new rsgroup name
1700 * @throws IOException if a remote or network exception occurs
1702 CompletableFuture<Void> renameRSGroup(String oldName, String newName);
1705 * Update RSGroup configuration
1706 * @param groupName the group name
1707 * @param configuration new configuration of the group name to be set
1708 * @throws IOException if a remote or network exception occurs
1710 CompletableFuture<Void> updateRSGroupConfig(String groupName, Map<String, String> configuration);
1713 * Retrieve recent online records from HMaster / RegionServers.
1714 * Examples include slow/large RPC logs, balancer decisions by master.
1716 * @param serverNames servers to retrieve records from, useful in case of records maintained
1717 * by RegionServer as we can select specific server. In case of servertype=MASTER, logs will
1718 * only come from the currently active master.
1719 * @param logType string representing type of log records
1720 * @param serverType enum for server type: HMaster or RegionServer
1721 * @param limit put a limit to list of records that server should send in response
1722 * @param filterParams additional filter params
1723 * @return Log entries representing online records from servers
1725 CompletableFuture<List<LogEntry>> getLogEntries(Set<ServerName> serverNames, String logType,
1726 ServerType serverType, int limit, Map<String, Object> filterParams);