001/** 002 * 003 * Licensed to the Apache Software Foundation (ASF) under one 004 * or more contributor license agreements. See the NOTICE file 005 * distributed with this work for additional information 006 * regarding copyright ownership. The ASF licenses this file 007 * to you under the Apache License, Version 2.0 (the 008 * "License"); you may not use this file except in compliance 009 * with the License. You may obtain a copy of the License at 010 * 011 * http://www.apache.org/licenses/LICENSE-2.0 012 * 013 * Unless required by applicable law or agreed to in writing, software 014 * distributed under the License is distributed on an "AS IS" BASIS, 015 * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. 016 * See the License for the specific language governing permissions and 017 * limitations under the License. 018 */ 019package org.apache.hadoop.hbase.regionserver; 020 021import java.io.IOException; 022import java.util.List; 023import java.util.concurrent.ConcurrentMap; 024 025import org.apache.hadoop.hbase.Abortable; 026import org.apache.hadoop.hbase.Server; 027import org.apache.hadoop.hbase.client.RegionInfo; 028import org.apache.hadoop.hbase.client.locking.EntityLock; 029import org.apache.hadoop.hbase.executor.ExecutorService; 030import org.apache.hadoop.hbase.ipc.RpcServerInterface; 031import org.apache.hadoop.hbase.quotas.RegionServerRpcQuotaManager; 032import org.apache.hadoop.hbase.quotas.RegionServerSpaceQuotaManager; 033import org.apache.hadoop.hbase.regionserver.compactions.CompactionRequester; 034import org.apache.hadoop.hbase.regionserver.throttle.ThroughputController; 035import org.apache.hadoop.hbase.wal.WAL; 036import org.apache.yetus.audience.InterfaceAudience; 037import org.apache.zookeeper.KeeperException; 038 039import org.apache.hadoop.hbase.shaded.protobuf.generated.RegionServerStatusProtos.RegionStateTransition.TransitionCode; 040 041import com.google.protobuf.Service; 042 043/** 044 * A curated subset of services provided by {@link HRegionServer}. 045 * For use internally only. Passed to Managers, Services and Chores so can pass less-than-a 046 * full-on HRegionServer at test-time. Be judicious adding API. Changes cause ripples through 047 * the code base. 048 */ 049@InterfaceAudience.Private 050public interface RegionServerServices extends Server, MutableOnlineRegions, FavoredNodesForRegion { 051 052 /** @return the WAL for a particular region. Pass null for getting the 053 * default (common) WAL */ 054 WAL getWAL(RegionInfo regionInfo) throws IOException; 055 056 /** @return the List of WALs that are used by this server 057 * Doesn't include the meta WAL 058 */ 059 List<WAL> getWALs() throws IOException; 060 061 /** 062 * @return Implementation of {@link FlushRequester} or null. Usually it will not be null unless 063 * during intialization. 064 */ 065 FlushRequester getFlushRequester(); 066 067 /** 068 * @return Implementation of {@link CompactionRequester} or null. Usually it will not be null 069 * unless during intialization. 070 */ 071 CompactionRequester getCompactionRequestor(); 072 073 /** 074 * @return the RegionServerAccounting for this Region Server 075 */ 076 RegionServerAccounting getRegionServerAccounting(); 077 078 /** 079 * @return RegionServer's instance of {@link RegionServerRpcQuotaManager} 080 */ 081 RegionServerRpcQuotaManager getRegionServerRpcQuotaManager(); 082 083 /** 084 * @return RegionServer's instance of {@link SecureBulkLoadManager} 085 */ 086 SecureBulkLoadManager getSecureBulkLoadManager(); 087 088 /** 089 * @return RegionServer's instance of {@link RegionServerSpaceQuotaManager} 090 */ 091 RegionServerSpaceQuotaManager getRegionServerSpaceQuotaManager(); 092 093 /** 094 * Context for postOpenDeployTasks(). 095 */ 096 class PostOpenDeployContext { 097 private final HRegion region; 098 private final long masterSystemTime; 099 100 @InterfaceAudience.Private 101 public PostOpenDeployContext(HRegion region, long masterSystemTime) { 102 this.region = region; 103 this.masterSystemTime = masterSystemTime; 104 } 105 public HRegion getRegion() { 106 return region; 107 } 108 public long getMasterSystemTime() { 109 return masterSystemTime; 110 } 111 } 112 113 /** 114 * Tasks to perform after region open to complete deploy of region on 115 * regionserver 116 * 117 * @param context the context 118 * @throws KeeperException 119 * @throws IOException 120 */ 121 void postOpenDeployTasks(final PostOpenDeployContext context) throws KeeperException, IOException; 122 123 class RegionStateTransitionContext { 124 private final TransitionCode code; 125 private final long openSeqNum; 126 private final long masterSystemTime; 127 private final RegionInfo[] hris; 128 129 @InterfaceAudience.Private 130 public RegionStateTransitionContext(TransitionCode code, long openSeqNum, long masterSystemTime, 131 RegionInfo... hris) { 132 this.code = code; 133 this.openSeqNum = openSeqNum; 134 this.masterSystemTime = masterSystemTime; 135 this.hris = hris; 136 } 137 public TransitionCode getCode() { 138 return code; 139 } 140 public long getOpenSeqNum() { 141 return openSeqNum; 142 } 143 public long getMasterSystemTime() { 144 return masterSystemTime; 145 } 146 public RegionInfo[] getHris() { 147 return hris; 148 } 149 } 150 151 /** 152 * Notify master that a handler requests to change a region state 153 */ 154 boolean reportRegionStateTransition(final RegionStateTransitionContext context); 155 156 /** 157 * Returns a reference to the region server's RPC server 158 */ 159 RpcServerInterface getRpcServer(); 160 161 /** 162 * Get the regions that are currently being opened or closed in the RS 163 * @return map of regions in transition in this RS 164 */ 165 ConcurrentMap<byte[], Boolean> getRegionsInTransitionInRS(); 166 167 /** 168 * @return The RegionServer's "Leases" service 169 */ 170 Leases getLeases(); 171 172 /** 173 * @return hbase executor service 174 */ 175 ExecutorService getExecutorService(); 176 177 /** 178 * Only required for "old" log replay; if it's removed, remove this. 179 * @return The RegionServer's NonceManager 180 */ 181 ServerNonceManager getNonceManager(); 182 183 /** 184 * Registers a new protocol buffer {@link Service} subclass as a coprocessor endpoint to be 185 * available for handling 186 * @param service the {@code Service} subclass instance to expose as a coprocessor endpoint 187 * @return {@code true} if the registration was successful, {@code false} 188 */ 189 boolean registerService(Service service); 190 191 /** 192 * @return heap memory manager instance 193 */ 194 HeapMemoryManager getHeapMemoryManager(); 195 196 /** 197 * @return the max compaction pressure of all stores on this regionserver. The value should be 198 * greater than or equal to 0.0, and any value greater than 1.0 means we enter the 199 * emergency state that some stores have too many store files. 200 * @see org.apache.hadoop.hbase.regionserver.Store#getCompactionPressure() 201 */ 202 double getCompactionPressure(); 203 204 /** 205 * @return the controller to avoid flush too fast 206 */ 207 ThroughputController getFlushThroughputController(); 208 209 /** 210 * @return the flush pressure of all stores on this regionserver. The value should be greater than 211 * or equal to 0.0, and any value greater than 1.0 means we enter the emergency state that 212 * global memstore size already exceeds lower limit. 213 */ 214 @Deprecated 215 double getFlushPressure(); 216 217 /** 218 * @return the metrics tracker for the region server 219 */ 220 MetricsRegionServer getMetrics(); 221 222 /** 223 * Master based locks on namespaces/tables/regions. 224 */ 225 EntityLock regionLock(List<RegionInfo> regionInfos, String description, 226 Abortable abort) throws IOException; 227 228 /** 229 * Unassign the given region from the current regionserver and assign it randomly. Could still be 230 * assigned to us. This is used to solve some tough problems for which you need to reset the state 231 * of a region. For example, if you hit FileNotFound exception and want to refresh the store file 232 * list. 233 * <p> 234 * See HBASE-17712 for more details. 235 */ 236 void unassign(byte[] regionName) throws IOException; 237 238 /** 239 * @return True if cluster is up; false if cluster is not up (we are shutting down). 240 */ 241 boolean isClusterUp(); 242}