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 046 private TimeBasedLimiter() { 047 if ( 048 FixedIntervalRateLimiter.class.getName().equals( 049 conf.getClass(RateLimiter.QUOTA_RATE_LIMITER_CONF_KEY, AverageIntervalRateLimiter.class) 050 .getName()) 051 ) { 052 long refillInterval = conf.getLong(FixedIntervalRateLimiter.RATE_LIMITER_REFILL_INTERVAL_MS, 053 RateLimiter.DEFAULT_TIME_UNIT); 054 reqsLimiter = new FixedIntervalRateLimiter(refillInterval); 055 reqSizeLimiter = new FixedIntervalRateLimiter(refillInterval); 056 writeReqsLimiter = new FixedIntervalRateLimiter(refillInterval); 057 writeSizeLimiter = new FixedIntervalRateLimiter(refillInterval); 058 readReqsLimiter = new FixedIntervalRateLimiter(refillInterval); 059 readSizeLimiter = new FixedIntervalRateLimiter(refillInterval); 060 reqCapacityUnitLimiter = new FixedIntervalRateLimiter(refillInterval); 061 writeCapacityUnitLimiter = new FixedIntervalRateLimiter(refillInterval); 062 readCapacityUnitLimiter = new FixedIntervalRateLimiter(refillInterval); 063 } else { 064 reqsLimiter = new AverageIntervalRateLimiter(); 065 reqSizeLimiter = new AverageIntervalRateLimiter(); 066 writeReqsLimiter = new AverageIntervalRateLimiter(); 067 writeSizeLimiter = new AverageIntervalRateLimiter(); 068 readReqsLimiter = new AverageIntervalRateLimiter(); 069 readSizeLimiter = new AverageIntervalRateLimiter(); 070 reqCapacityUnitLimiter = new AverageIntervalRateLimiter(); 071 writeCapacityUnitLimiter = new AverageIntervalRateLimiter(); 072 readCapacityUnitLimiter = new AverageIntervalRateLimiter(); 073 } 074 } 075 076 static QuotaLimiter fromThrottle(final Throttle throttle) { 077 TimeBasedLimiter limiter = new TimeBasedLimiter(); 078 boolean isBypass = true; 079 if (throttle.hasReqNum()) { 080 setFromTimedQuota(limiter.reqsLimiter, throttle.getReqNum()); 081 isBypass = false; 082 } 083 084 if (throttle.hasReqSize()) { 085 setFromTimedQuota(limiter.reqSizeLimiter, throttle.getReqSize()); 086 isBypass = false; 087 } 088 089 if (throttle.hasWriteNum()) { 090 setFromTimedQuota(limiter.writeReqsLimiter, throttle.getWriteNum()); 091 isBypass = false; 092 } 093 094 if (throttle.hasWriteSize()) { 095 setFromTimedQuota(limiter.writeSizeLimiter, throttle.getWriteSize()); 096 isBypass = false; 097 } 098 099 if (throttle.hasReadNum()) { 100 setFromTimedQuota(limiter.readReqsLimiter, throttle.getReadNum()); 101 isBypass = false; 102 } 103 104 if (throttle.hasReadSize()) { 105 setFromTimedQuota(limiter.readSizeLimiter, throttle.getReadSize()); 106 isBypass = false; 107 } 108 109 if (throttle.hasReqCapacityUnit()) { 110 setFromTimedQuota(limiter.reqCapacityUnitLimiter, throttle.getReqCapacityUnit()); 111 isBypass = false; 112 } 113 114 if (throttle.hasWriteCapacityUnit()) { 115 setFromTimedQuota(limiter.writeCapacityUnitLimiter, throttle.getWriteCapacityUnit()); 116 isBypass = false; 117 } 118 119 if (throttle.hasReadCapacityUnit()) { 120 setFromTimedQuota(limiter.readCapacityUnitLimiter, throttle.getReadCapacityUnit()); 121 isBypass = false; 122 } 123 return isBypass ? NoopQuotaLimiter.get() : limiter; 124 } 125 126 public void update(final TimeBasedLimiter other) { 127 reqsLimiter.update(other.reqsLimiter); 128 reqSizeLimiter.update(other.reqSizeLimiter); 129 writeReqsLimiter.update(other.writeReqsLimiter); 130 writeSizeLimiter.update(other.writeSizeLimiter); 131 readReqsLimiter.update(other.readReqsLimiter); 132 readSizeLimiter.update(other.readSizeLimiter); 133 reqCapacityUnitLimiter.update(other.reqCapacityUnitLimiter); 134 writeCapacityUnitLimiter.update(other.writeCapacityUnitLimiter); 135 readCapacityUnitLimiter.update(other.readCapacityUnitLimiter); 136 } 137 138 private static void setFromTimedQuota(final RateLimiter limiter, final TimedQuota timedQuota) { 139 limiter.set(timedQuota.getSoftLimit(), ProtobufUtil.toTimeUnit(timedQuota.getTimeUnit())); 140 } 141 142 @Override 143 public void checkQuota(long writeReqs, long estimateWriteSize, long readReqs, 144 long estimateReadSize, long estimateWriteCapacityUnit, long estimateReadCapacityUnit) 145 throws RpcThrottlingException { 146 long waitInterval = reqsLimiter.getWaitIntervalMs(writeReqs + readReqs); 147 if (waitInterval > 0) { 148 RpcThrottlingException.throwNumRequestsExceeded(waitInterval); 149 } 150 waitInterval = reqSizeLimiter.getWaitIntervalMs(estimateWriteSize + estimateReadSize); 151 if (waitInterval > 0) { 152 RpcThrottlingException.throwRequestSizeExceeded(waitInterval); 153 } 154 waitInterval = reqCapacityUnitLimiter 155 .getWaitIntervalMs(estimateWriteCapacityUnit + estimateReadCapacityUnit); 156 if (waitInterval > 0) { 157 RpcThrottlingException.throwRequestCapacityUnitExceeded(waitInterval); 158 } 159 160 if (estimateWriteSize > 0) { 161 waitInterval = writeReqsLimiter.getWaitIntervalMs(writeReqs); 162 if (waitInterval > 0) { 163 RpcThrottlingException.throwNumWriteRequestsExceeded(waitInterval); 164 } 165 waitInterval = writeSizeLimiter.getWaitIntervalMs(estimateWriteSize); 166 if (waitInterval > 0) { 167 RpcThrottlingException.throwWriteSizeExceeded(waitInterval); 168 } 169 waitInterval = writeCapacityUnitLimiter.getWaitIntervalMs(estimateWriteCapacityUnit); 170 if (waitInterval > 0) { 171 RpcThrottlingException.throwWriteCapacityUnitExceeded(waitInterval); 172 } 173 } 174 175 if (estimateReadSize > 0) { 176 waitInterval = readReqsLimiter.getWaitIntervalMs(readReqs); 177 if (waitInterval > 0) { 178 RpcThrottlingException.throwNumReadRequestsExceeded(waitInterval); 179 } 180 waitInterval = readSizeLimiter.getWaitIntervalMs(estimateReadSize); 181 if (waitInterval > 0) { 182 RpcThrottlingException.throwReadSizeExceeded(waitInterval); 183 } 184 waitInterval = readCapacityUnitLimiter.getWaitIntervalMs(estimateReadCapacityUnit); 185 if (waitInterval > 0) { 186 RpcThrottlingException.throwReadCapacityUnitExceeded(waitInterval); 187 } 188 } 189 } 190 191 @Override 192 public void grabQuota(long writeReqs, long writeSize, long readReqs, long readSize, 193 long writeCapacityUnit, long readCapacityUnit) { 194 assert writeSize != 0 || readSize != 0; 195 196 reqsLimiter.consume(writeReqs + readReqs); 197 reqSizeLimiter.consume(writeSize + readSize); 198 199 if (writeSize > 0) { 200 writeReqsLimiter.consume(writeReqs); 201 writeSizeLimiter.consume(writeSize); 202 } 203 if (readSize > 0) { 204 readReqsLimiter.consume(readReqs); 205 readSizeLimiter.consume(readSize); 206 } 207 if (writeCapacityUnit > 0) { 208 reqCapacityUnitLimiter.consume(writeCapacityUnit); 209 writeCapacityUnitLimiter.consume(writeCapacityUnit); 210 } 211 if (readCapacityUnit > 0) { 212 reqCapacityUnitLimiter.consume(readCapacityUnit); 213 readCapacityUnitLimiter.consume(readCapacityUnit); 214 } 215 } 216 217 @Override 218 public void consumeWrite(final long size, long capacityUnit) { 219 reqSizeLimiter.consume(size); 220 writeSizeLimiter.consume(size); 221 reqCapacityUnitLimiter.consume(capacityUnit); 222 writeCapacityUnitLimiter.consume(capacityUnit); 223 } 224 225 @Override 226 public void consumeRead(final long size, long capacityUnit) { 227 reqSizeLimiter.consume(size); 228 readSizeLimiter.consume(size); 229 reqCapacityUnitLimiter.consume(capacityUnit); 230 readCapacityUnitLimiter.consume(capacityUnit); 231 } 232 233 @Override 234 public boolean isBypass() { 235 return false; 236 } 237 238 @Override 239 public long getWriteAvailable() { 240 return writeSizeLimiter.getAvailable(); 241 } 242 243 @Override 244 public long getReadAvailable() { 245 return readSizeLimiter.getAvailable(); 246 } 247 248 @Override 249 public long getReadLimit() { 250 return Math.min(readSizeLimiter.getLimit(), reqSizeLimiter.getLimit()); 251 } 252 253 @Override 254 public String toString() { 255 StringBuilder builder = new StringBuilder(); 256 builder.append("TimeBasedLimiter("); 257 if (!reqsLimiter.isBypass()) { 258 builder.append("reqs=" + reqsLimiter); 259 } 260 if (!reqSizeLimiter.isBypass()) { 261 builder.append(" resSize=" + reqSizeLimiter); 262 } 263 if (!writeReqsLimiter.isBypass()) { 264 builder.append(" writeReqs=" + writeReqsLimiter); 265 } 266 if (!writeSizeLimiter.isBypass()) { 267 builder.append(" writeSize=" + writeSizeLimiter); 268 } 269 if (!readReqsLimiter.isBypass()) { 270 builder.append(" readReqs=" + readReqsLimiter); 271 } 272 if (!readSizeLimiter.isBypass()) { 273 builder.append(" readSize=" + readSizeLimiter); 274 } 275 if (!reqCapacityUnitLimiter.isBypass()) { 276 builder.append(" reqCapacityUnit=" + reqCapacityUnitLimiter); 277 } 278 if (!writeCapacityUnitLimiter.isBypass()) { 279 builder.append(" writeCapacityUnit=" + writeCapacityUnitLimiter); 280 } 281 if (!readCapacityUnitLimiter.isBypass()) { 282 builder.append(" readCapacityUnit=" + readCapacityUnitLimiter); 283 } 284 builder.append(')'); 285 return builder.toString(); 286 } 287}