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.quotas; 019 020import org.apache.hadoop.conf.Configuration; 021import org.apache.hadoop.hbase.HBaseConfiguration; 022import org.apache.yetus.audience.InterfaceAudience; 023import org.apache.yetus.audience.InterfaceStability; 024 025import org.apache.hadoop.hbase.shaded.protobuf.ProtobufUtil; 026import org.apache.hadoop.hbase.shaded.protobuf.generated.QuotaProtos.Throttle; 027import org.apache.hadoop.hbase.shaded.protobuf.generated.QuotaProtos.TimedQuota; 028 029/** 030 * Simple time based limiter that checks the quota Throttle 031 */ 032@InterfaceAudience.Private 033@InterfaceStability.Evolving 034public class TimeBasedLimiter implements QuotaLimiter { 035 private static final Configuration conf = HBaseConfiguration.create(); 036 private RateLimiter reqsLimiter = null; 037 private RateLimiter reqSizeLimiter = null; 038 private RateLimiter writeReqsLimiter = null; 039 private RateLimiter writeSizeLimiter = null; 040 private RateLimiter readReqsLimiter = null; 041 private RateLimiter readSizeLimiter = null; 042 private RateLimiter reqCapacityUnitLimiter = null; 043 private RateLimiter writeCapacityUnitLimiter = null; 044 private RateLimiter readCapacityUnitLimiter = null; 045 private RateLimiter atomicReqLimiter = null; 046 private RateLimiter atomicReadSizeLimiter = null; 047 private RateLimiter atomicWriteSizeLimiter = null; 048 private RateLimiter reqHandlerUsageTimeLimiter = null; 049 050 private TimeBasedLimiter() { 051 if ( 052 FixedIntervalRateLimiter.class.getName().equals( 053 conf.getClass(RateLimiter.QUOTA_RATE_LIMITER_CONF_KEY, AverageIntervalRateLimiter.class) 054 .getName()) 055 ) { 056 long refillInterval = conf.getLong(FixedIntervalRateLimiter.RATE_LIMITER_REFILL_INTERVAL_MS, 057 RateLimiter.DEFAULT_TIME_UNIT); 058 reqsLimiter = new FixedIntervalRateLimiter(refillInterval); 059 reqSizeLimiter = new FixedIntervalRateLimiter(refillInterval); 060 writeReqsLimiter = new FixedIntervalRateLimiter(refillInterval); 061 writeSizeLimiter = new FixedIntervalRateLimiter(refillInterval); 062 readReqsLimiter = new FixedIntervalRateLimiter(refillInterval); 063 readSizeLimiter = new FixedIntervalRateLimiter(refillInterval); 064 reqCapacityUnitLimiter = new FixedIntervalRateLimiter(refillInterval); 065 writeCapacityUnitLimiter = new FixedIntervalRateLimiter(refillInterval); 066 readCapacityUnitLimiter = new FixedIntervalRateLimiter(refillInterval); 067 atomicReqLimiter = new FixedIntervalRateLimiter(refillInterval); 068 atomicReadSizeLimiter = new FixedIntervalRateLimiter(refillInterval); 069 atomicWriteSizeLimiter = new FixedIntervalRateLimiter(refillInterval); 070 reqHandlerUsageTimeLimiter = new FixedIntervalRateLimiter(refillInterval); 071 } else { 072 reqsLimiter = new AverageIntervalRateLimiter(); 073 reqSizeLimiter = new AverageIntervalRateLimiter(); 074 writeReqsLimiter = new AverageIntervalRateLimiter(); 075 writeSizeLimiter = new AverageIntervalRateLimiter(); 076 readReqsLimiter = new AverageIntervalRateLimiter(); 077 readSizeLimiter = new AverageIntervalRateLimiter(); 078 reqCapacityUnitLimiter = new AverageIntervalRateLimiter(); 079 writeCapacityUnitLimiter = new AverageIntervalRateLimiter(); 080 readCapacityUnitLimiter = new AverageIntervalRateLimiter(); 081 atomicReqLimiter = new AverageIntervalRateLimiter(); 082 atomicReadSizeLimiter = new AverageIntervalRateLimiter(); 083 atomicWriteSizeLimiter = new AverageIntervalRateLimiter(); 084 reqHandlerUsageTimeLimiter = new AverageIntervalRateLimiter(); 085 } 086 } 087 088 static QuotaLimiter fromThrottle(final Throttle throttle) { 089 TimeBasedLimiter limiter = new TimeBasedLimiter(); 090 boolean isBypass = true; 091 if (throttle.hasReqNum()) { 092 setFromTimedQuota(limiter.reqsLimiter, throttle.getReqNum()); 093 isBypass = false; 094 } 095 096 if (throttle.hasReqSize()) { 097 setFromTimedQuota(limiter.reqSizeLimiter, throttle.getReqSize()); 098 isBypass = false; 099 } 100 101 if (throttle.hasWriteNum()) { 102 setFromTimedQuota(limiter.writeReqsLimiter, throttle.getWriteNum()); 103 isBypass = false; 104 } 105 106 if (throttle.hasWriteSize()) { 107 setFromTimedQuota(limiter.writeSizeLimiter, throttle.getWriteSize()); 108 isBypass = false; 109 } 110 111 if (throttle.hasReadNum()) { 112 setFromTimedQuota(limiter.readReqsLimiter, throttle.getReadNum()); 113 isBypass = false; 114 } 115 116 if (throttle.hasReadSize()) { 117 setFromTimedQuota(limiter.readSizeLimiter, throttle.getReadSize()); 118 isBypass = false; 119 } 120 121 if (throttle.hasReqCapacityUnit()) { 122 setFromTimedQuota(limiter.reqCapacityUnitLimiter, throttle.getReqCapacityUnit()); 123 isBypass = false; 124 } 125 126 if (throttle.hasWriteCapacityUnit()) { 127 setFromTimedQuota(limiter.writeCapacityUnitLimiter, throttle.getWriteCapacityUnit()); 128 isBypass = false; 129 } 130 131 if (throttle.hasReadCapacityUnit()) { 132 setFromTimedQuota(limiter.readCapacityUnitLimiter, throttle.getReadCapacityUnit()); 133 isBypass = false; 134 } 135 136 if (throttle.hasAtomicReqNum()) { 137 setFromTimedQuota(limiter.atomicReqLimiter, throttle.getAtomicReqNum()); 138 isBypass = false; 139 } 140 141 if (throttle.hasAtomicReadSize()) { 142 setFromTimedQuota(limiter.atomicReadSizeLimiter, throttle.getAtomicReadSize()); 143 isBypass = false; 144 } 145 146 if (throttle.hasAtomicWriteSize()) { 147 setFromTimedQuota(limiter.atomicWriteSizeLimiter, throttle.getAtomicWriteSize()); 148 isBypass = false; 149 } 150 151 if (throttle.hasReqHandlerUsageMs()) { 152 setFromTimedQuota(limiter.reqHandlerUsageTimeLimiter, throttle.getReqHandlerUsageMs()); 153 isBypass = false; 154 } 155 156 return isBypass ? NoopQuotaLimiter.get() : limiter; 157 } 158 159 public void update(final TimeBasedLimiter other) { 160 reqsLimiter.update(other.reqsLimiter); 161 reqSizeLimiter.update(other.reqSizeLimiter); 162 writeReqsLimiter.update(other.writeReqsLimiter); 163 writeSizeLimiter.update(other.writeSizeLimiter); 164 readReqsLimiter.update(other.readReqsLimiter); 165 readSizeLimiter.update(other.readSizeLimiter); 166 reqCapacityUnitLimiter.update(other.reqCapacityUnitLimiter); 167 writeCapacityUnitLimiter.update(other.writeCapacityUnitLimiter); 168 readCapacityUnitLimiter.update(other.readCapacityUnitLimiter); 169 atomicReqLimiter.update(other.atomicReqLimiter); 170 atomicReadSizeLimiter.update(other.atomicReadSizeLimiter); 171 atomicWriteSizeLimiter.update(other.atomicWriteSizeLimiter); 172 reqHandlerUsageTimeLimiter.update(other.reqHandlerUsageTimeLimiter); 173 } 174 175 private static void setFromTimedQuota(final RateLimiter limiter, final TimedQuota timedQuota) { 176 limiter.set(timedQuota.getSoftLimit(), ProtobufUtil.toTimeUnit(timedQuota.getTimeUnit())); 177 } 178 179 @Override 180 public void checkQuota(long writeReqs, long estimateWriteSize, long readReqs, 181 long estimateReadSize, long estimateWriteCapacityUnit, long estimateReadCapacityUnit, 182 boolean isAtomic, long estimatedReqHandlerUsageTimeMs) throws RpcThrottlingException { 183 long waitInterval = reqsLimiter.getWaitIntervalMs(writeReqs + readReqs); 184 if (waitInterval > 0) { 185 RpcThrottlingException.throwNumRequestsExceeded(waitInterval); 186 } 187 waitInterval = reqSizeLimiter.getWaitIntervalMs(estimateWriteSize + estimateReadSize); 188 if (waitInterval > 0) { 189 RpcThrottlingException.throwRequestSizeExceeded(waitInterval); 190 } 191 waitInterval = reqCapacityUnitLimiter 192 .getWaitIntervalMs(estimateWriteCapacityUnit + estimateReadCapacityUnit); 193 if (waitInterval > 0) { 194 RpcThrottlingException.throwRequestCapacityUnitExceeded(waitInterval); 195 } 196 if (isAtomic) { 197 waitInterval = atomicReqLimiter.getWaitIntervalMs(writeReqs + readReqs); 198 if (waitInterval > 0) { 199 RpcThrottlingException.throwAtomicRequestNumberExceeded(waitInterval); 200 } 201 } 202 203 if (estimateWriteSize > 0) { 204 waitInterval = writeReqsLimiter.getWaitIntervalMs(writeReqs); 205 if (waitInterval > 0) { 206 RpcThrottlingException.throwNumWriteRequestsExceeded(waitInterval); 207 } 208 waitInterval = writeSizeLimiter.getWaitIntervalMs(estimateWriteSize); 209 if (waitInterval > 0) { 210 RpcThrottlingException.throwWriteSizeExceeded(waitInterval); 211 } 212 waitInterval = writeCapacityUnitLimiter.getWaitIntervalMs(estimateWriteCapacityUnit); 213 if (waitInterval > 0) { 214 RpcThrottlingException.throwWriteCapacityUnitExceeded(waitInterval); 215 } 216 if (isAtomic) { 217 waitInterval = atomicWriteSizeLimiter.getWaitIntervalMs(writeReqs); 218 if (waitInterval > 0) { 219 RpcThrottlingException.throwAtomicWriteSizeExceeded(waitInterval); 220 } 221 } 222 } 223 224 if (estimateReadSize > 0) { 225 waitInterval = readReqsLimiter.getWaitIntervalMs(readReqs); 226 if (waitInterval > 0) { 227 RpcThrottlingException.throwNumReadRequestsExceeded(waitInterval); 228 } 229 waitInterval = readSizeLimiter.getWaitIntervalMs(estimateReadSize); 230 if (waitInterval > 0) { 231 RpcThrottlingException.throwReadSizeExceeded(waitInterval); 232 } 233 waitInterval = readCapacityUnitLimiter.getWaitIntervalMs(estimateReadCapacityUnit); 234 if (waitInterval > 0) { 235 RpcThrottlingException.throwReadCapacityUnitExceeded(waitInterval); 236 } 237 if (isAtomic) { 238 waitInterval = atomicReadSizeLimiter.getWaitIntervalMs(writeReqs + readReqs); 239 if (waitInterval > 0) { 240 RpcThrottlingException.throwAtomicReadSizeExceeded(waitInterval); 241 } 242 } 243 } 244 waitInterval = reqHandlerUsageTimeLimiter.getWaitIntervalMs(estimatedReqHandlerUsageTimeMs); 245 if (waitInterval > 0) { 246 RpcThrottlingException.throwRequestHandlerUsageTimeExceeded(waitInterval); 247 } 248 } 249 250 @Override 251 public void grabQuota(long writeReqs, long writeSize, long readReqs, long readSize, 252 long writeCapacityUnit, long readCapacityUnit, boolean isAtomic, 253 long estimateHandlerThreadUsageMs) { 254 assert writeSize != 0 || readSize != 0; 255 256 reqsLimiter.consume(writeReqs + readReqs); 257 reqSizeLimiter.consume(writeSize + readSize); 258 259 if (writeSize > 0) { 260 writeReqsLimiter.consume(writeReqs); 261 writeSizeLimiter.consume(writeSize); 262 } 263 if (readSize > 0) { 264 readReqsLimiter.consume(readReqs); 265 readSizeLimiter.consume(readSize); 266 } 267 if (writeCapacityUnit > 0) { 268 reqCapacityUnitLimiter.consume(writeCapacityUnit); 269 writeCapacityUnitLimiter.consume(writeCapacityUnit); 270 } 271 if (readCapacityUnit > 0) { 272 reqCapacityUnitLimiter.consume(readCapacityUnit); 273 readCapacityUnitLimiter.consume(readCapacityUnit); 274 } 275 if (isAtomic) { 276 atomicReqLimiter.consume(writeReqs + readReqs); 277 if (readSize > 0) { 278 atomicReadSizeLimiter.consume(readSize); 279 } 280 if (writeSize > 0) { 281 atomicWriteSizeLimiter.consume(writeSize); 282 } 283 } 284 reqHandlerUsageTimeLimiter.consume(estimateHandlerThreadUsageMs); 285 } 286 287 @Override 288 public void consumeWrite(final long size, long capacityUnit, boolean isAtomic) { 289 reqSizeLimiter.consume(size); 290 writeSizeLimiter.consume(size); 291 reqCapacityUnitLimiter.consume(capacityUnit); 292 writeCapacityUnitLimiter.consume(capacityUnit); 293 if (isAtomic) { 294 atomicWriteSizeLimiter.consume(size); 295 } 296 } 297 298 @Override 299 public void consumeRead(final long size, long capacityUnit, boolean isAtomic) { 300 reqSizeLimiter.consume(size); 301 readSizeLimiter.consume(size); 302 reqCapacityUnitLimiter.consume(capacityUnit); 303 readCapacityUnitLimiter.consume(capacityUnit); 304 if (isAtomic) { 305 atomicReadSizeLimiter.consume(size); 306 } 307 } 308 309 @Override 310 public void consumeTime(final long handlerMillisUsed) { 311 reqHandlerUsageTimeLimiter.consume(handlerMillisUsed); 312 } 313 314 @Override 315 public boolean isBypass() { 316 return false; 317 } 318 319 @Override 320 public long getWriteAvailable() { 321 return writeSizeLimiter.getAvailable(); 322 } 323 324 @Override 325 public long getRequestNumLimit() { 326 long readAndWriteLimit = readReqsLimiter.getLimit() + writeReqsLimiter.getLimit(); 327 328 if (readAndWriteLimit < 0) { // handle overflow 329 readAndWriteLimit = Long.MAX_VALUE; 330 } 331 332 return Math.min(reqsLimiter.getLimit(), readAndWriteLimit); 333 } 334 335 @Override 336 public long getReadNumLimit() { 337 return readReqsLimiter.getLimit(); 338 } 339 340 @Override 341 public long getWriteNumLimit() { 342 return writeReqsLimiter.getLimit(); 343 } 344 345 @Override 346 public long getReadAvailable() { 347 return readSizeLimiter.getAvailable(); 348 } 349 350 @Override 351 public long getReadLimit() { 352 return Math.min(readSizeLimiter.getLimit(), reqSizeLimiter.getLimit()); 353 } 354 355 @Override 356 public long getWriteLimit() { 357 return Math.min(writeSizeLimiter.getLimit(), reqSizeLimiter.getLimit()); 358 } 359 360 @Override 361 public String toString() { 362 StringBuilder builder = new StringBuilder(); 363 builder.append("TimeBasedLimiter("); 364 if (!reqsLimiter.isBypass()) { 365 builder.append("reqs=" + reqsLimiter); 366 } 367 if (!reqSizeLimiter.isBypass()) { 368 builder.append(" resSize=" + reqSizeLimiter); 369 } 370 if (!writeReqsLimiter.isBypass()) { 371 builder.append(" writeReqs=" + writeReqsLimiter); 372 } 373 if (!writeSizeLimiter.isBypass()) { 374 builder.append(" writeSize=" + writeSizeLimiter); 375 } 376 if (!readReqsLimiter.isBypass()) { 377 builder.append(" readReqs=" + readReqsLimiter); 378 } 379 if (!readSizeLimiter.isBypass()) { 380 builder.append(" readSize=" + readSizeLimiter); 381 } 382 if (!reqCapacityUnitLimiter.isBypass()) { 383 builder.append(" reqCapacityUnit=" + reqCapacityUnitLimiter); 384 } 385 if (!writeCapacityUnitLimiter.isBypass()) { 386 builder.append(" writeCapacityUnit=" + writeCapacityUnitLimiter); 387 } 388 if (!readCapacityUnitLimiter.isBypass()) { 389 builder.append(" readCapacityUnit=" + readCapacityUnitLimiter); 390 } 391 if (!atomicReqLimiter.isBypass()) { 392 builder.append(" atomicReqLimiter=" + atomicReqLimiter); 393 } 394 if (!atomicReadSizeLimiter.isBypass()) { 395 builder.append(" atomicReadSizeLimiter=" + atomicReadSizeLimiter); 396 } 397 if (!atomicWriteSizeLimiter.isBypass()) { 398 builder.append(" atomicWriteSizeLimiter=" + atomicWriteSizeLimiter); 399 } 400 if (!reqHandlerUsageTimeLimiter.isBypass()) { 401 builder.append(" reqHandlerUsageTimeLimiter=" + reqHandlerUsageTimeLimiter); 402 } 403 builder.append(')'); 404 return builder.toString(); 405 } 406}