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.ipc; 019 020import java.util.Queue; 021import java.util.concurrent.atomic.AtomicInteger; 022import org.apache.hadoop.conf.Configuration; 023import org.apache.hadoop.hbase.Abortable; 024import org.apache.hadoop.hbase.HBaseInterfaceAudience; 025import org.apache.hadoop.hbase.conf.ConfigurationObserver; 026import org.apache.yetus.audience.InterfaceAudience; 027import org.apache.yetus.audience.InterfaceStability; 028import org.slf4j.Logger; 029import org.slf4j.LoggerFactory; 030 031import org.apache.hbase.thirdparty.com.google.protobuf.Message; 032 033import org.apache.hadoop.hbase.shaded.protobuf.generated.ClientProtos.Action; 034import org.apache.hadoop.hbase.shaded.protobuf.generated.ClientProtos.MultiRequest; 035import org.apache.hadoop.hbase.shaded.protobuf.generated.ClientProtos.MutateRequest; 036import org.apache.hadoop.hbase.shaded.protobuf.generated.ClientProtos.RegionAction; 037import org.apache.hadoop.hbase.shaded.protobuf.generated.ClientProtos.ScanRequest; 038import org.apache.hadoop.hbase.shaded.protobuf.generated.RPCProtos.RequestHeader; 039import org.apache.hadoop.hbase.shaded.protobuf.generated.RegionServerStatusProtos; 040 041/** 042 * RPC Executor that uses different queues for reads and writes. With the options to use different 043 * queues/executors for gets and scans. Each handler has its own queue and there is no stealing. 044 */ 045@InterfaceAudience.LimitedPrivate({ HBaseInterfaceAudience.COPROC, HBaseInterfaceAudience.PHOENIX }) 046@InterfaceStability.Evolving 047public class RWQueueRpcExecutor extends RpcExecutor { 048 private static final Logger LOG = LoggerFactory.getLogger(RWQueueRpcExecutor.class); 049 050 public static final String CALL_QUEUE_READ_SHARE_CONF_KEY = 051 "hbase.ipc.server.callqueue.read.ratio"; 052 public static final String CALL_QUEUE_SCAN_SHARE_CONF_KEY = 053 "hbase.ipc.server.callqueue.scan.ratio"; 054 055 private final QueueBalancer writeBalancer; 056 private final QueueBalancer readBalancer; 057 private final QueueBalancer scanBalancer; 058 private final int writeHandlersCount; 059 private final int readHandlersCount; 060 private final int scanHandlersCount; 061 private final int numWriteQueues; 062 private final int numReadQueues; 063 private final int numScanQueues; 064 065 private final AtomicInteger activeWriteHandlerCount = new AtomicInteger(0); 066 private final AtomicInteger activeReadHandlerCount = new AtomicInteger(0); 067 private final AtomicInteger activeScanHandlerCount = new AtomicInteger(0); 068 069 public RWQueueRpcExecutor(final String name, final int handlerCount, final int maxQueueLength, 070 final PriorityFunction priority, final Configuration conf, final Abortable abortable) { 071 super(name, handlerCount, maxQueueLength, priority, conf, abortable); 072 073 float callqReadShare = getReadShare(conf); 074 float callqScanShare = getScanShare(conf); 075 076 numWriteQueues = calcNumWriters(this.numCallQueues, callqReadShare); 077 writeHandlersCount = Math.max(numWriteQueues, calcNumWriters(handlerCount, callqReadShare)); 078 079 int readQueues = calcNumReaders(this.numCallQueues, callqReadShare); 080 int readHandlers = Math.max(readQueues, calcNumReaders(handlerCount, callqReadShare)); 081 082 int scanHandlers = Math.max(0, (int) Math.floor(readHandlers * callqScanShare)); 083 int scanQueues = 084 scanHandlers > 0 ? Math.max(1, (int) Math.floor(readQueues * callqScanShare)) : 0; 085 086 if (scanQueues > 0) { 087 // if scanQueues > 0, the handler count of read should > 0, then we make readQueues >= 1 088 readQueues = Math.max(1, readQueues - scanQueues); 089 readHandlers -= scanHandlers; 090 } else { 091 scanQueues = 0; 092 scanHandlers = 0; 093 } 094 095 numReadQueues = readQueues; 096 readHandlersCount = readHandlers; 097 numScanQueues = scanQueues; 098 scanHandlersCount = scanHandlers; 099 100 initializeQueues(numWriteQueues + numReadQueues + numScanQueues); 101 102 this.writeBalancer = getBalancer(name, conf, queues.subList(0, numWriteQueues)); 103 this.readBalancer = 104 getBalancer(name, conf, queues.subList(numWriteQueues, numWriteQueues + numReadQueues)); 105 this.scanBalancer = numScanQueues > 0 106 ? getBalancer(name, conf, 107 queues.subList(numWriteQueues + numReadQueues, 108 numWriteQueues + numReadQueues + numScanQueues)) 109 : null; 110 111 LOG.info(getName() + " writeQueues=" + numWriteQueues + " writeHandlers=" + writeHandlersCount 112 + " readQueues=" + numReadQueues + " readHandlers=" + readHandlersCount + " scanQueues=" 113 + numScanQueues + " scanHandlers=" + scanHandlersCount); 114 } 115 116 @Override 117 protected int computeNumCallQueues(final int handlerCount, final float callQueuesHandlersFactor) { 118 // at least 1 read queue and 1 write queue 119 return Math.max(2, (int) Math.round(handlerCount * callQueuesHandlersFactor)); 120 } 121 122 @Override 123 protected void startHandlers(final int port) { 124 startHandlers(".write", writeHandlersCount, queues, 0, numWriteQueues, port, 125 activeWriteHandlerCount); 126 startHandlers(".read", readHandlersCount, queues, numWriteQueues, numReadQueues, port, 127 activeReadHandlerCount); 128 if (numScanQueues > 0) { 129 startHandlers(".scan", scanHandlersCount, queues, numWriteQueues + numReadQueues, 130 numScanQueues, port, activeScanHandlerCount); 131 } 132 } 133 134 @Override 135 public boolean dispatch(final CallRunner callTask) { 136 RpcCall call = callTask.getRpcCall(); 137 return dispatchTo(isWriteRequest(call.getHeader(), call.getParam()), 138 shouldDispatchToScanQueue(callTask), callTask); 139 } 140 141 protected boolean dispatchTo(boolean toWriteQueue, boolean toScanQueue, 142 final CallRunner callTask) { 143 int queueIndex; 144 if (toWriteQueue) { 145 queueIndex = writeBalancer.getNextQueue(callTask); 146 } else if (toScanQueue) { 147 queueIndex = numWriteQueues + numReadQueues + scanBalancer.getNextQueue(callTask); 148 } else { 149 queueIndex = numWriteQueues + readBalancer.getNextQueue(callTask); 150 } 151 Queue<CallRunner> queue = queues.get(queueIndex); 152 if (queue.size() >= currentQueueLimit) { 153 return false; 154 } 155 return queue.offer(callTask); 156 } 157 158 @Override 159 public int getWriteQueueLength() { 160 int length = 0; 161 for (int i = 0; i < numWriteQueues; i++) { 162 length += queues.get(i).size(); 163 } 164 return length; 165 } 166 167 @Override 168 public int getReadQueueLength() { 169 int length = 0; 170 for (int i = numWriteQueues; i < (numWriteQueues + numReadQueues); i++) { 171 length += queues.get(i).size(); 172 } 173 return length; 174 } 175 176 @Override 177 public int getScanQueueLength() { 178 int length = 0; 179 for (int i = numWriteQueues + numReadQueues; i 180 < (numWriteQueues + numReadQueues + numScanQueues); i++) { 181 length += queues.get(i).size(); 182 } 183 return length; 184 } 185 186 @Override 187 public int getActiveHandlerCount() { 188 return activeWriteHandlerCount.get() + activeReadHandlerCount.get() 189 + activeScanHandlerCount.get(); 190 } 191 192 @Override 193 public int getActiveWriteHandlerCount() { 194 return activeWriteHandlerCount.get(); 195 } 196 197 @Override 198 public int getActiveReadHandlerCount() { 199 return activeReadHandlerCount.get(); 200 } 201 202 @Override 203 public int getActiveScanHandlerCount() { 204 return activeScanHandlerCount.get(); 205 } 206 207 protected boolean isWriteRequest(final RequestHeader header, final Message param) { 208 // TODO: Is there a better way to do this? 209 if (param instanceof MultiRequest) { 210 MultiRequest multi = (MultiRequest) param; 211 for (RegionAction regionAction : multi.getRegionActionList()) { 212 for (Action action : regionAction.getActionList()) { 213 if (action.hasMutation()) { 214 return true; 215 } 216 } 217 } 218 } 219 if (param instanceof MutateRequest) { 220 return true; 221 } 222 // Below here are methods for master. It's a pretty brittle version of this. 223 // Not sure that master actually needs a read/write queue since 90% of requests to 224 // master are writing to status or changing the meta table. 225 // All other read requests are admin generated and can be processed whenever. 226 // However changing that would require a pretty drastic change and should be done for 227 // the next major release and not as a fix for HBASE-14239 228 if (param instanceof RegionServerStatusProtos.ReportRegionStateTransitionRequest) { 229 return true; 230 } 231 if (param instanceof RegionServerStatusProtos.RegionServerStartupRequest) { 232 return true; 233 } 234 if (param instanceof RegionServerStatusProtos.RegionServerReportRequest) { 235 return true; 236 } 237 return false; 238 } 239 240 QueueBalancer getWriteBalancer() { 241 return writeBalancer; 242 } 243 244 QueueBalancer getReadBalancer() { 245 return readBalancer; 246 } 247 248 QueueBalancer getScanBalancer() { 249 return scanBalancer; 250 } 251 252 private boolean isScanRequest(final RequestHeader header, final Message param) { 253 return param instanceof ScanRequest; 254 } 255 256 protected boolean shouldDispatchToScanQueue(final CallRunner task) { 257 RpcCall call = task.getRpcCall(); 258 return numScanQueues > 0 && isScanRequest(call.getHeader(), call.getParam()); 259 } 260 261 protected float getReadShare(final Configuration conf) { 262 return conf.getFloat(CALL_QUEUE_READ_SHARE_CONF_KEY, 0); 263 } 264 265 protected float getScanShare(final Configuration conf) { 266 return conf.getFloat(CALL_QUEUE_SCAN_SHARE_CONF_KEY, 0); 267 } 268 269 /* 270 * Calculate the number of writers based on the "total count" and the read share. You'll get at 271 * least one writer. 272 */ 273 private static int calcNumWriters(final int count, final float readShare) { 274 return Math.max(1, count - Math.max(1, (int) Math.round(count * readShare))); 275 } 276 277 /* 278 * Calculate the number of readers based on the "total count" and the read share. You'll get at 279 * least one reader. 280 */ 281 private static int calcNumReaders(final int count, final float readShare) { 282 return count - calcNumWriters(count, readShare); 283 } 284 285 @Override 286 public void onConfigurationChange(Configuration conf) { 287 super.onConfigurationChange(conf); 288 propagateBalancerConfigChange(writeBalancer, conf); 289 propagateBalancerConfigChange(readBalancer, conf); 290 propagateBalancerConfigChange(scanBalancer, conf); 291 } 292 293 private void propagateBalancerConfigChange(QueueBalancer balancer, Configuration conf) { 294 if (balancer instanceof ConfigurationObserver) { 295 ((ConfigurationObserver) balancer).onConfigurationChange(conf); 296 } 297 } 298 299 protected int getNumScanQueues() { 300 return numScanQueues; 301 } 302}