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