001/**
002 * Licensed to the Apache Software Foundation (ASF) under one
003 * or more contributor license agreements.  See the NOTICE file
004 * distributed with this work for additional information
005 * regarding copyright ownership.  The ASF licenses this file
006 * to you under the Apache License, Version 2.0 (the
007 * "License"); you may not use this file except in compliance
008 * with the License.  You may obtain a copy of the License at
009 *
010 *     http://www.apache.org/licenses/LICENSE-2.0
011 *
012 * Unless required by applicable law or agreed to in writing, software
013 * distributed under the License is distributed on an "AS IS" BASIS,
014 * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
015 * See the License for the specific language governing permissions and
016 * limitations under the License.
017 */
018package org.apache.hadoop.hbase.master;
019
020import com.google.protobuf.Service;
021
022import java.io.IOException;
023import java.util.List;
024
025import org.apache.hadoop.hbase.Server;
026import org.apache.hadoop.hbase.ServerName;
027import org.apache.hadoop.hbase.TableDescriptors;
028import org.apache.hadoop.hbase.TableName;
029import org.apache.hadoop.hbase.TableNotDisabledException;
030import org.apache.hadoop.hbase.TableNotFoundException;
031import org.apache.hadoop.hbase.client.ColumnFamilyDescriptor;
032import org.apache.hadoop.hbase.client.MasterSwitchType;
033import org.apache.hadoop.hbase.client.RegionInfo;
034import org.apache.hadoop.hbase.client.TableDescriptor;
035import org.apache.hadoop.hbase.executor.ExecutorService;
036import org.apache.hadoop.hbase.favored.FavoredNodesManager;
037import org.apache.hadoop.hbase.master.assignment.AssignmentManager;
038import org.apache.hadoop.hbase.master.locking.LockManager;
039import org.apache.hadoop.hbase.master.normalizer.RegionNormalizer;
040import org.apache.hadoop.hbase.master.procedure.MasterProcedureEnv;
041import org.apache.hadoop.hbase.master.replication.ReplicationPeerManager;
042import org.apache.hadoop.hbase.master.snapshot.SnapshotManager;
043import org.apache.hadoop.hbase.procedure.MasterProcedureManagerHost;
044import org.apache.hadoop.hbase.procedure2.LockedResource;
045import org.apache.hadoop.hbase.procedure2.Procedure;
046import org.apache.hadoop.hbase.procedure2.ProcedureEvent;
047import org.apache.hadoop.hbase.procedure2.ProcedureExecutor;
048import org.apache.hadoop.hbase.quotas.MasterQuotaManager;
049import org.apache.hadoop.hbase.replication.ReplicationException;
050import org.apache.hadoop.hbase.replication.ReplicationPeerConfig;
051import org.apache.hadoop.hbase.replication.ReplicationPeerDescription;
052import org.apache.yetus.audience.InterfaceAudience;
053
054import org.apache.hbase.thirdparty.com.google.common.annotations.VisibleForTesting;
055
056/**
057 * A curated subset of services provided by {@link HMaster}.
058 * For use internally only. Passed to Managers, Services and Chores so can pass less-than-a
059 * full-on HMaster at test-time. Be judicious adding API. Changes cause ripples through
060 * the code base.
061 */
062@InterfaceAudience.Private
063public interface MasterServices extends Server {
064  /**
065   * @return the underlying snapshot manager
066   */
067  SnapshotManager getSnapshotManager();
068
069  /**
070   * @return the underlying MasterProcedureManagerHost
071   */
072  MasterProcedureManagerHost getMasterProcedureManagerHost();
073
074  /**
075   * @return Master's instance of {@link ClusterSchema}
076   */
077  ClusterSchema getClusterSchema();
078
079  /**
080   * @return Master's instance of the {@link AssignmentManager}
081   */
082  AssignmentManager getAssignmentManager();
083
084  /**
085   * @return Master's filesystem {@link MasterFileSystem} utility class.
086   */
087  MasterFileSystem getMasterFileSystem();
088
089  /**
090   * @return Master's WALs {@link MasterWalManager} utility class.
091   */
092  MasterWalManager getMasterWalManager();
093
094  /**
095   * @return Master's {@link ServerManager} instance.
096   */
097  ServerManager getServerManager();
098
099  /**
100   * @return Master's instance of {@link ExecutorService}
101   */
102  ExecutorService getExecutorService();
103
104  /**
105   * @return Master's instance of {@link TableStateManager}
106   */
107  TableStateManager getTableStateManager();
108
109  /**
110   * @return Master's instance of {@link MasterCoprocessorHost}
111   */
112  MasterCoprocessorHost getMasterCoprocessorHost();
113
114  /**
115   * @return Master's instance of {@link MasterQuotaManager}
116   */
117  MasterQuotaManager getMasterQuotaManager();
118
119  /**
120   * @return Master's instance of {@link RegionNormalizer}
121   */
122  RegionNormalizer getRegionNormalizer();
123
124  /**
125   * @return Master's instance of {@link CatalogJanitor}
126   */
127  CatalogJanitor getCatalogJanitor();
128
129  /**
130   * @return Master's instance of {@link ProcedureExecutor}
131   */
132  ProcedureExecutor<MasterProcedureEnv> getMasterProcedureExecutor();
133
134  /**
135   * @return Tripped when Master has finished initialization.
136   */
137  @VisibleForTesting
138  public ProcedureEvent<?> getInitializedEvent();
139
140  /**
141   * @return Master's instance of {@link MetricsMaster}
142   */
143  MetricsMaster getMasterMetrics();
144
145  /**
146   * Check table is modifiable; i.e. exists and is offline.
147   * @param tableName Name of table to check.
148   * @throws TableNotDisabledException
149   * @throws TableNotFoundException
150   * @throws IOException
151   */
152  // We actually throw the exceptions mentioned in the
153  void checkTableModifiable(final TableName tableName)
154      throws IOException, TableNotFoundException, TableNotDisabledException;
155
156  /**
157   * Create a table using the given table definition.
158   * @param desc The table definition
159   * @param splitKeys Starting row keys for the initial table regions.  If null
160   * @param nonceGroup
161   * @param nonce
162   *     a single region is created.
163   */
164  long createTable(
165      final TableDescriptor desc,
166      final byte[][] splitKeys,
167      final long nonceGroup,
168      final long nonce) throws IOException;
169
170  /**
171   * Create a system table using the given table definition.
172   * @param tableDescriptor The system table definition
173   *     a single region is created.
174   */
175  long createSystemTable(final TableDescriptor tableDescriptor) throws IOException;
176
177  /**
178   * Delete a table
179   * @param tableName The table name
180   * @param nonceGroup
181   * @param nonce
182   * @throws IOException
183   */
184  long deleteTable(
185      final TableName tableName,
186      final long nonceGroup,
187      final long nonce) throws IOException;
188
189  /**
190   * Truncate a table
191   * @param tableName The table name
192   * @param preserveSplits True if the splits should be preserved
193   * @param nonceGroup
194   * @param nonce
195   * @throws IOException
196   */
197  public long truncateTable(
198      final TableName tableName,
199      final boolean preserveSplits,
200      final long nonceGroup,
201      final long nonce) throws IOException;
202
203  /**
204   * Modify the descriptor of an existing table
205   * @param tableName The table name
206   * @param descriptor The updated table descriptor
207   * @param nonceGroup
208   * @param nonce
209   * @throws IOException
210   */
211  long modifyTable(
212      final TableName tableName,
213      final TableDescriptor descriptor,
214      final long nonceGroup,
215      final long nonce)
216      throws IOException;
217
218  /**
219   * Enable an existing table
220   * @param tableName The table name
221   * @param nonceGroup
222   * @param nonce
223   * @throws IOException
224   */
225  long enableTable(
226      final TableName tableName,
227      final long nonceGroup,
228      final long nonce) throws IOException;
229
230  /**
231   * Disable an existing table
232   * @param tableName The table name
233   * @param nonceGroup
234   * @param nonce
235   * @throws IOException
236   */
237  long disableTable(
238      final TableName tableName,
239      final long nonceGroup,
240      final long nonce) throws IOException;
241
242
243  /**
244   * Add a new column to an existing table
245   * @param tableName The table name
246   * @param column The column definition
247   * @param nonceGroup
248   * @param nonce
249   * @throws IOException
250   */
251  long addColumn(
252      final TableName tableName,
253      final ColumnFamilyDescriptor column,
254      final long nonceGroup,
255      final long nonce)
256      throws IOException;
257
258  /**
259   * Modify the column descriptor of an existing column in an existing table
260   * @param tableName The table name
261   * @param descriptor The updated column definition
262   * @param nonceGroup
263   * @param nonce
264   * @throws IOException
265   */
266  long modifyColumn(
267      final TableName tableName,
268      final ColumnFamilyDescriptor descriptor,
269      final long nonceGroup,
270      final long nonce)
271      throws IOException;
272
273  /**
274   * Delete a column from an existing table
275   * @param tableName The table name
276   * @param columnName The column name
277   * @param nonceGroup
278   * @param nonce
279   * @throws IOException
280   */
281  long deleteColumn(
282      final TableName tableName,
283      final byte[] columnName,
284      final long nonceGroup,
285      final long nonce)
286      throws IOException;
287
288  /**
289   * Merge regions in a table.
290   * @param regionsToMerge daughter regions to merge
291   * @param forcible whether to force to merge even two regions are not adjacent
292   * @param nonceGroup used to detect duplicate
293   * @param nonce used to detect duplicate
294   * @return  procedure Id
295   * @throws IOException
296   */
297  long mergeRegions(
298      final RegionInfo[] regionsToMerge,
299      final boolean forcible,
300      final long nonceGroup,
301      final long nonce) throws IOException;
302
303  /**
304   * Split a region.
305   * @param regionInfo region to split
306   * @param splitRow split point
307   * @param nonceGroup used to detect duplicate
308   * @param nonce used to detect duplicate
309   * @return  procedure Id
310   * @throws IOException
311   */
312  long splitRegion(
313      final RegionInfo regionInfo,
314      final byte [] splitRow,
315      final long nonceGroup,
316      final long nonce) throws IOException;
317
318  /**
319   * @return Return table descriptors implementation.
320   */
321  TableDescriptors getTableDescriptors();
322
323  /**
324   * Registers a new protocol buffer {@link Service} subclass as a master coprocessor endpoint.
325   *
326   * <p>
327   * Only a single instance may be registered for a given {@link Service} subclass (the
328   * instances are keyed on {@link com.google.protobuf.Descriptors.ServiceDescriptor#getFullName()}.
329   * After the first registration, subsequent calls with the same service name will fail with
330   * a return value of {@code false}.
331   * </p>
332   * @param instance the {@code Service} subclass instance to expose as a coprocessor endpoint
333   * @return {@code true} if the registration was successful, {@code false}
334   * otherwise
335   */
336  boolean registerService(Service instance);
337
338  /**
339   * @return true if master is the active one
340   */
341  boolean isActiveMaster();
342
343  /**
344   * @return true if master is initialized
345   */
346  boolean isInitialized();
347
348  /**
349   * @return true if master is in maintanceMode
350   * @throws IOException if the inquiry failed due to an IO problem
351   */
352  boolean isInMaintenanceMode();
353
354  /**
355   * Abort a procedure.
356   * @param procId ID of the procedure
357   * @param mayInterruptIfRunning if the proc completed at least one step, should it be aborted?
358   * @return true if aborted, false if procedure already completed or does not exist
359   * @throws IOException
360   */
361  public boolean abortProcedure(final long procId, final boolean mayInterruptIfRunning)
362      throws IOException;
363
364  /**
365   * Get procedures
366   * @return procedure list
367   * @throws IOException
368   */
369  public List<Procedure<?>> getProcedures() throws IOException;
370
371  /**
372   * Get locks
373   * @return lock list
374   * @throws IOException
375   */
376  public List<LockedResource> getLocks() throws IOException;
377
378  /**
379   * Get list of table descriptors by namespace
380   * @param name namespace name
381   * @return descriptors
382   * @throws IOException
383   */
384  public List<TableDescriptor> listTableDescriptorsByNamespace(String name) throws IOException;
385
386  /**
387   * Get list of table names by namespace
388   * @param name namespace name
389   * @return table names
390   * @throws IOException
391   */
392  public List<TableName> listTableNamesByNamespace(String name) throws IOException;
393
394  /**
395   * @param table the table for which last successful major compaction time is queried
396   * @return the timestamp of the last successful major compaction for the passed table,
397   * or 0 if no HFile resulting from a major compaction exists
398   * @throws IOException
399   */
400  public long getLastMajorCompactionTimestamp(TableName table) throws IOException;
401
402  /**
403   * @param regionName
404   * @return the timestamp of the last successful major compaction for the passed region
405   * or 0 if no HFile resulting from a major compaction exists
406   * @throws IOException
407   */
408  public long getLastMajorCompactionTimestampForRegion(byte[] regionName) throws IOException;
409
410  /**
411   * @return load balancer
412   */
413  public LoadBalancer getLoadBalancer();
414
415  boolean isSplitOrMergeEnabled(MasterSwitchType switchType);
416
417  /**
418   * @return Favored Nodes Manager
419   */
420  public FavoredNodesManager getFavoredNodesManager();
421
422  /**
423   * Add a new replication peer for replicating data to slave cluster
424   * @param peerId a short name that identifies the peer
425   * @param peerConfig configuration for the replication slave cluster
426   * @param enabled peer state, true if ENABLED and false if DISABLED
427   */
428  long addReplicationPeer(String peerId, ReplicationPeerConfig peerConfig, boolean enabled)
429      throws ReplicationException, IOException;
430
431  /**
432   * Removes a peer and stops the replication
433   * @param peerId a short name that identifies the peer
434   */
435  long removeReplicationPeer(String peerId) throws ReplicationException, IOException;
436
437  /**
438   * Restart the replication stream to the specified peer
439   * @param peerId a short name that identifies the peer
440   */
441  long enableReplicationPeer(String peerId) throws ReplicationException, IOException;
442
443  /**
444   * Stop the replication stream to the specified peer
445   * @param peerId a short name that identifies the peer
446   */
447  long disableReplicationPeer(String peerId) throws ReplicationException, IOException;
448
449  /**
450   * Returns the configured ReplicationPeerConfig for the specified peer
451   * @param peerId a short name that identifies the peer
452   * @return ReplicationPeerConfig for the peer
453   */
454  ReplicationPeerConfig getReplicationPeerConfig(String peerId) throws ReplicationException,
455      IOException;
456
457  /**
458   * Returns the {@link ReplicationPeerManager}.
459   */
460  ReplicationPeerManager getReplicationPeerManager();
461
462  /**
463   * Update the peerConfig for the specified peer
464   * @param peerId a short name that identifies the peer
465   * @param peerConfig new config for the peer
466   */
467  long updateReplicationPeerConfig(String peerId, ReplicationPeerConfig peerConfig)
468      throws ReplicationException, IOException;
469
470  /**
471   * Return a list of replication peers.
472   * @param regex The regular expression to match peer id
473   * @return a list of replication peers description
474   */
475  List<ReplicationPeerDescription> listReplicationPeers(String regex) throws ReplicationException,
476      IOException;
477
478  /**
479   * @return {@link LockManager} to lock namespaces/tables/regions.
480   */
481  LockManager getLockManager();
482
483  public String getRegionServerVersion(final ServerName sn);
484
485  /**
486   * Called when a new RegionServer is added to the cluster.
487   * Checks if new server has a newer version than any existing server and will move system tables
488   * there if so.
489   */
490  public void checkIfShouldMoveSystemRegionAsync();
491
492  String getClientIdAuditPrefix();
493
494  /**
495   * @return True if cluster is up; false if cluster is not up (we are shutting down).
496   */
497  boolean isClusterUp();
498
499  /**
500   * @return return null if current is zk-based WAL splitting
501   */
502  default SplitWALManager getSplitWALManager(){
503    return null;
504  }
505
506  /**
507   * Execute region plans with throttling
508   * @param plans to execute
509   * @return succeeded plans
510   */
511  List<RegionPlan> executeRegionPlansWithThrottling(List<RegionPlan> plans);
512
513  /**
514   * Run the ReplicationBarrierChore.
515   */
516  void runReplicationBarrierCleaner();
517}