HBASE-22403 Balance in RSGroup should consider throttling and a failure affects the...
[hbase.git] / hbase-server / src / main / java / org / apache / hadoop / hbase / master / MasterServices.java
blob41cec5cfb232877d7aaece2b43862bd9515a8938
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.master;
20 import com.google.protobuf.Service;
22 import java.io.IOException;
23 import java.util.List;
25 import org.apache.hadoop.hbase.Server;
26 import org.apache.hadoop.hbase.ServerName;
27 import org.apache.hadoop.hbase.TableDescriptors;
28 import org.apache.hadoop.hbase.TableName;
29 import org.apache.hadoop.hbase.TableNotDisabledException;
30 import org.apache.hadoop.hbase.TableNotFoundException;
31 import org.apache.hadoop.hbase.client.ColumnFamilyDescriptor;
32 import org.apache.hadoop.hbase.client.MasterSwitchType;
33 import org.apache.hadoop.hbase.client.RegionInfo;
34 import org.apache.hadoop.hbase.client.TableDescriptor;
35 import org.apache.hadoop.hbase.executor.ExecutorService;
36 import org.apache.hadoop.hbase.favored.FavoredNodesManager;
37 import org.apache.hadoop.hbase.master.assignment.AssignmentManager;
38 import org.apache.hadoop.hbase.master.locking.LockManager;
39 import org.apache.hadoop.hbase.master.normalizer.RegionNormalizer;
40 import org.apache.hadoop.hbase.master.procedure.MasterProcedureEnv;
41 import org.apache.hadoop.hbase.master.replication.ReplicationPeerManager;
42 import org.apache.hadoop.hbase.master.replication.SyncReplicationReplayWALManager;
43 import org.apache.hadoop.hbase.master.snapshot.SnapshotManager;
44 import org.apache.hadoop.hbase.procedure.MasterProcedureManagerHost;
45 import org.apache.hadoop.hbase.procedure2.LockedResource;
46 import org.apache.hadoop.hbase.procedure2.Procedure;
47 import org.apache.hadoop.hbase.procedure2.ProcedureEvent;
48 import org.apache.hadoop.hbase.procedure2.ProcedureExecutor;
49 import org.apache.hadoop.hbase.quotas.MasterQuotaManager;
50 import org.apache.hadoop.hbase.replication.ReplicationException;
51 import org.apache.hadoop.hbase.replication.ReplicationPeerConfig;
52 import org.apache.hadoop.hbase.replication.ReplicationPeerDescription;
53 import org.apache.hadoop.hbase.replication.SyncReplicationState;
54 import org.apache.hadoop.hbase.security.access.AccessChecker;
55 import org.apache.hadoop.hbase.security.access.ZKPermissionWatcher;
56 import org.apache.yetus.audience.InterfaceAudience;
58 import org.apache.hbase.thirdparty.com.google.common.annotations.VisibleForTesting;
60 /**
61 * A curated subset of services provided by {@link HMaster}.
62 * For use internally only. Passed to Managers, Services and Chores so can pass less-than-a
63 * full-on HMaster at test-time. Be judicious adding API. Changes cause ripples through
64 * the code base.
66 @InterfaceAudience.Private
67 public interface MasterServices extends Server {
68 /**
69 * @return the underlying snapshot manager
71 SnapshotManager getSnapshotManager();
73 /**
74 * @return the underlying MasterProcedureManagerHost
76 MasterProcedureManagerHost getMasterProcedureManagerHost();
78 /**
79 * @return Master's instance of {@link ClusterSchema}
81 ClusterSchema getClusterSchema();
83 /**
84 * @return Master's instance of the {@link AssignmentManager}
86 AssignmentManager getAssignmentManager();
88 /**
89 * @return Master's filesystem {@link MasterFileSystem} utility class.
91 MasterFileSystem getMasterFileSystem();
93 /**
94 * @return Master's WALs {@link MasterWalManager} utility class.
96 MasterWalManager getMasterWalManager();
98 /**
99 * @return Master's {@link ServerManager} instance.
101 ServerManager getServerManager();
104 * @return Master's instance of {@link ExecutorService}
106 ExecutorService getExecutorService();
109 * @return Master's instance of {@link TableStateManager}
111 TableStateManager getTableStateManager();
114 * @return Master's instance of {@link MasterCoprocessorHost}
116 MasterCoprocessorHost getMasterCoprocessorHost();
119 * @return Master's instance of {@link MasterQuotaManager}
121 MasterQuotaManager getMasterQuotaManager();
124 * @return Master's instance of {@link RegionNormalizer}
126 RegionNormalizer getRegionNormalizer();
129 * @return Master's instance of {@link CatalogJanitor}
131 CatalogJanitor getCatalogJanitor();
134 * @return Master's instance of {@link ProcedureExecutor}
136 ProcedureExecutor<MasterProcedureEnv> getMasterProcedureExecutor();
139 * @return Tripped when Master has finished initialization.
141 @VisibleForTesting
142 public ProcedureEvent<?> getInitializedEvent();
145 * @return Master's instance of {@link MetricsMaster}
147 MetricsMaster getMasterMetrics();
150 * Check table is modifiable; i.e. exists and is offline.
151 * @param tableName Name of table to check.
152 * @throws TableNotDisabledException
153 * @throws TableNotFoundException
154 * @throws IOException
156 // We actually throw the exceptions mentioned in the
157 void checkTableModifiable(final TableName tableName)
158 throws IOException, TableNotFoundException, TableNotDisabledException;
161 * Create a table using the given table definition.
162 * @param desc The table definition
163 * @param splitKeys Starting row keys for the initial table regions. If null
164 * @param nonceGroup
165 * @param nonce
166 * a single region is created.
168 long createTable(
169 final TableDescriptor desc,
170 final byte[][] splitKeys,
171 final long nonceGroup,
172 final long nonce) throws IOException;
175 * Create a system table using the given table definition.
176 * @param tableDescriptor The system table definition
177 * a single region is created.
179 long createSystemTable(final TableDescriptor tableDescriptor) throws IOException;
182 * Delete a table
183 * @param tableName The table name
184 * @param nonceGroup
185 * @param nonce
186 * @throws IOException
188 long deleteTable(
189 final TableName tableName,
190 final long nonceGroup,
191 final long nonce) throws IOException;
194 * Truncate a table
195 * @param tableName The table name
196 * @param preserveSplits True if the splits should be preserved
197 * @param nonceGroup
198 * @param nonce
199 * @throws IOException
201 public long truncateTable(
202 final TableName tableName,
203 final boolean preserveSplits,
204 final long nonceGroup,
205 final long nonce) throws IOException;
208 * Modify the descriptor of an existing table
209 * @param tableName The table name
210 * @param descriptor The updated table descriptor
211 * @param nonceGroup
212 * @param nonce
213 * @throws IOException
215 long modifyTable(
216 final TableName tableName,
217 final TableDescriptor descriptor,
218 final long nonceGroup,
219 final long nonce)
220 throws IOException;
223 * Enable an existing table
224 * @param tableName The table name
225 * @param nonceGroup
226 * @param nonce
227 * @throws IOException
229 long enableTable(
230 final TableName tableName,
231 final long nonceGroup,
232 final long nonce) throws IOException;
235 * Disable an existing table
236 * @param tableName The table name
237 * @param nonceGroup
238 * @param nonce
239 * @throws IOException
241 long disableTable(
242 final TableName tableName,
243 final long nonceGroup,
244 final long nonce) throws IOException;
248 * Add a new column to an existing table
249 * @param tableName The table name
250 * @param column The column definition
251 * @param nonceGroup
252 * @param nonce
253 * @throws IOException
255 long addColumn(
256 final TableName tableName,
257 final ColumnFamilyDescriptor column,
258 final long nonceGroup,
259 final long nonce)
260 throws IOException;
263 * Modify the column descriptor of an existing column in an existing table
264 * @param tableName The table name
265 * @param descriptor The updated column definition
266 * @param nonceGroup
267 * @param nonce
268 * @throws IOException
270 long modifyColumn(
271 final TableName tableName,
272 final ColumnFamilyDescriptor descriptor,
273 final long nonceGroup,
274 final long nonce)
275 throws IOException;
278 * Delete a column from an existing table
279 * @param tableName The table name
280 * @param columnName The column name
281 * @param nonceGroup
282 * @param nonce
283 * @throws IOException
285 long deleteColumn(
286 final TableName tableName,
287 final byte[] columnName,
288 final long nonceGroup,
289 final long nonce)
290 throws IOException;
293 * Merge regions in a table.
294 * @param regionsToMerge daughter regions to merge
295 * @param forcible whether to force to merge even two regions are not adjacent
296 * @param nonceGroup used to detect duplicate
297 * @param nonce used to detect duplicate
298 * @return procedure Id
299 * @throws IOException
301 long mergeRegions(
302 final RegionInfo[] regionsToMerge,
303 final boolean forcible,
304 final long nonceGroup,
305 final long nonce) throws IOException;
308 * Split a region.
309 * @param regionInfo region to split
310 * @param splitRow split point
311 * @param nonceGroup used to detect duplicate
312 * @param nonce used to detect duplicate
313 * @return procedure Id
314 * @throws IOException
316 long splitRegion(
317 final RegionInfo regionInfo,
318 final byte [] splitRow,
319 final long nonceGroup,
320 final long nonce) throws IOException;
323 * @return Return table descriptors implementation.
325 TableDescriptors getTableDescriptors();
328 * Registers a new protocol buffer {@link Service} subclass as a master coprocessor endpoint.
330 * <p>
331 * Only a single instance may be registered for a given {@link Service} subclass (the
332 * instances are keyed on {@link com.google.protobuf.Descriptors.ServiceDescriptor#getFullName()}.
333 * After the first registration, subsequent calls with the same service name will fail with
334 * a return value of {@code false}.
335 * </p>
336 * @param instance the {@code Service} subclass instance to expose as a coprocessor endpoint
337 * @return {@code true} if the registration was successful, {@code false}
338 * otherwise
340 boolean registerService(Service instance);
343 * @return true if master is the active one
345 boolean isActiveMaster();
348 * @return true if master is initialized
350 boolean isInitialized();
353 * @return true if master is in maintanceMode
354 * @throws IOException if the inquiry failed due to an IO problem
356 boolean isInMaintenanceMode();
359 * Abort a procedure.
360 * @param procId ID of the procedure
361 * @param mayInterruptIfRunning if the proc completed at least one step, should it be aborted?
362 * @return true if aborted, false if procedure already completed or does not exist
363 * @throws IOException
365 public boolean abortProcedure(final long procId, final boolean mayInterruptIfRunning)
366 throws IOException;
369 * Get procedures
370 * @return procedure list
371 * @throws IOException
373 public List<Procedure<?>> getProcedures() throws IOException;
376 * Get locks
377 * @return lock list
378 * @throws IOException
380 public List<LockedResource> getLocks() throws IOException;
383 * Get list of table descriptors by namespace
384 * @param name namespace name
385 * @return descriptors
386 * @throws IOException
388 public List<TableDescriptor> listTableDescriptorsByNamespace(String name) throws IOException;
391 * Get list of table names by namespace
392 * @param name namespace name
393 * @return table names
394 * @throws IOException
396 public List<TableName> listTableNamesByNamespace(String name) throws IOException;
399 * @param table the table for which last successful major compaction time is queried
400 * @return the timestamp of the last successful major compaction for the passed table,
401 * or 0 if no HFile resulting from a major compaction exists
402 * @throws IOException
404 public long getLastMajorCompactionTimestamp(TableName table) throws IOException;
407 * @param regionName
408 * @return the timestamp of the last successful major compaction for the passed region
409 * or 0 if no HFile resulting from a major compaction exists
410 * @throws IOException
412 public long getLastMajorCompactionTimestampForRegion(byte[] regionName) throws IOException;
415 * @return load balancer
417 public LoadBalancer getLoadBalancer();
419 boolean isSplitOrMergeEnabled(MasterSwitchType switchType);
422 * @return Favored Nodes Manager
424 public FavoredNodesManager getFavoredNodesManager();
427 * Add a new replication peer for replicating data to slave cluster
428 * @param peerId a short name that identifies the peer
429 * @param peerConfig configuration for the replication slave cluster
430 * @param enabled peer state, true if ENABLED and false if DISABLED
432 long addReplicationPeer(String peerId, ReplicationPeerConfig peerConfig, boolean enabled)
433 throws ReplicationException, IOException;
436 * Removes a peer and stops the replication
437 * @param peerId a short name that identifies the peer
439 long removeReplicationPeer(String peerId) throws ReplicationException, IOException;
442 * Restart the replication stream to the specified peer
443 * @param peerId a short name that identifies the peer
445 long enableReplicationPeer(String peerId) throws ReplicationException, IOException;
448 * Stop the replication stream to the specified peer
449 * @param peerId a short name that identifies the peer
451 long disableReplicationPeer(String peerId) throws ReplicationException, IOException;
454 * Returns the configured ReplicationPeerConfig for the specified peer
455 * @param peerId a short name that identifies the peer
456 * @return ReplicationPeerConfig for the peer
458 ReplicationPeerConfig getReplicationPeerConfig(String peerId) throws ReplicationException,
459 IOException;
462 * Returns the {@link ReplicationPeerManager}.
464 ReplicationPeerManager getReplicationPeerManager();
467 * Returns the {@link SyncReplicationReplayWALManager}.
469 SyncReplicationReplayWALManager getSyncReplicationReplayWALManager();
472 * Update the peerConfig for the specified peer
473 * @param peerId a short name that identifies the peer
474 * @param peerConfig new config for the peer
476 long updateReplicationPeerConfig(String peerId, ReplicationPeerConfig peerConfig)
477 throws ReplicationException, IOException;
480 * Return a list of replication peers.
481 * @param regex The regular expression to match peer id
482 * @return a list of replication peers description
484 List<ReplicationPeerDescription> listReplicationPeers(String regex) throws ReplicationException,
485 IOException;
488 * Set current cluster state for a synchronous replication peer.
489 * @param peerId a short name that identifies the peer
490 * @param clusterState state of current cluster
492 long transitReplicationPeerSyncReplicationState(String peerId, SyncReplicationState clusterState)
493 throws ReplicationException, IOException;
496 * @return {@link LockManager} to lock namespaces/tables/regions.
498 LockManager getLockManager();
500 public String getRegionServerVersion(final ServerName sn);
503 * Called when a new RegionServer is added to the cluster.
504 * Checks if new server has a newer version than any existing server and will move system tables
505 * there if so.
507 public void checkIfShouldMoveSystemRegionAsync();
509 String getClientIdAuditPrefix();
512 * @return True if cluster is up; false if cluster is not up (we are shutting down).
514 boolean isClusterUp();
517 * @return return null if current is zk-based WAL splitting
519 default SplitWALManager getSplitWALManager(){
520 return null;
524 * @return the {@link AccessChecker}
526 AccessChecker getAccessChecker();
529 * @return the {@link ZKPermissionWatcher}
531 ZKPermissionWatcher getZKPermissionWatcher();
534 * Execute region plans with throttling
535 * @param plans to execute
536 * @return succeeded plans
538 List<RegionPlan> executeRegionPlansWithThrottling(List<RegionPlan> plans);