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 */ 018 019package org.apache.hadoop.hbase.quotas; 020 021 022import org.apache.hadoop.conf.Configuration; 023import org.apache.hadoop.hbase.HBaseConfiguration; 024import org.apache.yetus.audience.InterfaceAudience; 025import org.apache.yetus.audience.InterfaceStability; 026import org.apache.hadoop.hbase.shaded.protobuf.ProtobufUtil; 027import org.apache.hadoop.hbase.shaded.protobuf.generated.QuotaProtos.Throttle; 028import org.apache.hadoop.hbase.shaded.protobuf.generated.QuotaProtos.TimedQuota; 029 030/** 031 * Simple time based limiter that checks the quota Throttle 032 */ 033@InterfaceAudience.Private 034@InterfaceStability.Evolving 035public class TimeBasedLimiter implements QuotaLimiter { 036 private static final Configuration conf = HBaseConfiguration.create(); 037 private RateLimiter reqsLimiter = null; 038 private RateLimiter reqSizeLimiter = null; 039 private RateLimiter writeReqsLimiter = null; 040 private RateLimiter writeSizeLimiter = null; 041 private RateLimiter readReqsLimiter = null; 042 private RateLimiter readSizeLimiter = null; 043 044 private TimeBasedLimiter() { 045 if (FixedIntervalRateLimiter.class.getName().equals( 046 conf.getClass(RateLimiter.QUOTA_RATE_LIMITER_CONF_KEY, AverageIntervalRateLimiter.class) 047 .getName())) { 048 reqsLimiter = new FixedIntervalRateLimiter(); 049 reqSizeLimiter = new FixedIntervalRateLimiter(); 050 writeReqsLimiter = new FixedIntervalRateLimiter(); 051 writeSizeLimiter = new FixedIntervalRateLimiter(); 052 readReqsLimiter = new FixedIntervalRateLimiter(); 053 readSizeLimiter = new FixedIntervalRateLimiter(); 054 } else { 055 reqsLimiter = new AverageIntervalRateLimiter(); 056 reqSizeLimiter = new AverageIntervalRateLimiter(); 057 writeReqsLimiter = new AverageIntervalRateLimiter(); 058 writeSizeLimiter = new AverageIntervalRateLimiter(); 059 readReqsLimiter = new AverageIntervalRateLimiter(); 060 readSizeLimiter = new AverageIntervalRateLimiter(); 061 } 062 } 063 064 static QuotaLimiter fromThrottle(final Throttle throttle) { 065 TimeBasedLimiter limiter = new TimeBasedLimiter(); 066 boolean isBypass = true; 067 if (throttle.hasReqNum()) { 068 setFromTimedQuota(limiter.reqsLimiter, throttle.getReqNum()); 069 isBypass = false; 070 } 071 072 if (throttle.hasReqSize()) { 073 setFromTimedQuota(limiter.reqSizeLimiter, throttle.getReqSize()); 074 isBypass = false; 075 } 076 077 if (throttle.hasWriteNum()) { 078 setFromTimedQuota(limiter.writeReqsLimiter, throttle.getWriteNum()); 079 isBypass = false; 080 } 081 082 if (throttle.hasWriteSize()) { 083 setFromTimedQuota(limiter.writeSizeLimiter, throttle.getWriteSize()); 084 isBypass = false; 085 } 086 087 if (throttle.hasReadNum()) { 088 setFromTimedQuota(limiter.readReqsLimiter, throttle.getReadNum()); 089 isBypass = false; 090 } 091 092 if (throttle.hasReadSize()) { 093 setFromTimedQuota(limiter.readSizeLimiter, throttle.getReadSize()); 094 isBypass = false; 095 } 096 return isBypass ? NoopQuotaLimiter.get() : limiter; 097 } 098 099 public void update(final TimeBasedLimiter other) { 100 reqsLimiter.update(other.reqsLimiter); 101 reqSizeLimiter.update(other.reqSizeLimiter); 102 writeReqsLimiter.update(other.writeReqsLimiter); 103 writeSizeLimiter.update(other.writeSizeLimiter); 104 readReqsLimiter.update(other.readReqsLimiter); 105 readSizeLimiter.update(other.readSizeLimiter); 106 } 107 108 private static void setFromTimedQuota(final RateLimiter limiter, final TimedQuota timedQuota) { 109 limiter.set(timedQuota.getSoftLimit(), ProtobufUtil.toTimeUnit(timedQuota.getTimeUnit())); 110 } 111 112 @Override 113 public void checkQuota(long writeReqs, long estimateWriteSize, long readReqs, 114 long estimateReadSize) throws RpcThrottlingException { 115 if (!reqsLimiter.canExecute(writeReqs + readReqs)) { 116 RpcThrottlingException.throwNumRequestsExceeded(reqsLimiter.waitInterval()); 117 } 118 if (!reqSizeLimiter.canExecute(estimateWriteSize + estimateReadSize)) { 119 RpcThrottlingException.throwRequestSizeExceeded( 120 reqSizeLimiter.waitInterval(estimateWriteSize + estimateReadSize)); 121 } 122 123 if (estimateWriteSize > 0) { 124 if (!writeReqsLimiter.canExecute(writeReqs)) { 125 RpcThrottlingException.throwNumWriteRequestsExceeded(writeReqsLimiter.waitInterval()); 126 } 127 if (!writeSizeLimiter.canExecute(estimateWriteSize)) { 128 RpcThrottlingException.throwWriteSizeExceeded( 129 writeSizeLimiter.waitInterval(estimateWriteSize)); 130 } 131 } 132 133 if (estimateReadSize > 0) { 134 if (!readReqsLimiter.canExecute(readReqs)) { 135 RpcThrottlingException.throwNumReadRequestsExceeded(readReqsLimiter.waitInterval()); 136 } 137 if (!readSizeLimiter.canExecute(estimateReadSize)) { 138 RpcThrottlingException.throwReadSizeExceeded( 139 readSizeLimiter.waitInterval(estimateReadSize)); 140 } 141 } 142 } 143 144 @Override 145 public void grabQuota(long writeReqs, long writeSize, long readReqs, long readSize) { 146 assert writeSize != 0 || readSize != 0; 147 148 reqsLimiter.consume(writeReqs + readReqs); 149 reqSizeLimiter.consume(writeSize + readSize); 150 151 if (writeSize > 0) { 152 writeReqsLimiter.consume(writeReqs); 153 writeSizeLimiter.consume(writeSize); 154 } 155 if (readSize > 0) { 156 readReqsLimiter.consume(readReqs); 157 readSizeLimiter.consume(readSize); 158 } 159 } 160 161 @Override 162 public void consumeWrite(final long size) { 163 reqSizeLimiter.consume(size); 164 writeSizeLimiter.consume(size); 165 } 166 167 @Override 168 public void consumeRead(final long size) { 169 reqSizeLimiter.consume(size); 170 readSizeLimiter.consume(size); 171 } 172 173 @Override 174 public boolean isBypass() { 175 return false; 176 } 177 178 @Override 179 public long getWriteAvailable() { 180 return writeSizeLimiter.getAvailable(); 181 } 182 183 @Override 184 public long getReadAvailable() { 185 return readSizeLimiter.getAvailable(); 186 } 187 188 @Override 189 public String toString() { 190 StringBuilder builder = new StringBuilder(); 191 builder.append("TimeBasedLimiter("); 192 if (!reqsLimiter.isBypass()) builder.append("reqs=" + reqsLimiter); 193 if (!reqSizeLimiter.isBypass()) builder.append(" resSize=" + reqSizeLimiter); 194 if (!writeReqsLimiter.isBypass()) builder.append(" writeReqs=" + writeReqsLimiter); 195 if (!writeSizeLimiter.isBypass()) builder.append(" writeSize=" + writeSizeLimiter); 196 if (!readReqsLimiter.isBypass()) builder.append(" readReqs=" + readReqsLimiter); 197 if (!readSizeLimiter.isBypass()) builder.append(" readSize=" + readSizeLimiter); 198 builder.append(')'); 199 return builder.toString(); 200 } 201}