1
2
3
4
5
6
7
8
9
10
11
12
13
14
15
16
17
18
19 package org.apache.hadoop.hbase.ipc;
20
21 import java.util.ArrayList;
22 import java.util.List;
23 import java.util.concurrent.BlockingQueue;
24 import java.util.concurrent.LinkedBlockingQueue;
25
26 import org.apache.commons.lang.ArrayUtils;
27 import org.apache.commons.logging.Log;
28 import org.apache.commons.logging.LogFactory;
29 import org.apache.hadoop.conf.Configuration;
30 import org.apache.hadoop.hbase.Abortable;
31 import org.apache.hadoop.hbase.HBaseInterfaceAudience;
32 import org.apache.hadoop.hbase.classification.InterfaceAudience;
33 import org.apache.hadoop.hbase.classification.InterfaceStability;
34 import org.apache.hadoop.hbase.protobuf.generated.ClientProtos.Action;
35 import org.apache.hadoop.hbase.protobuf.generated.ClientProtos.MultiRequest;
36 import org.apache.hadoop.hbase.protobuf.generated.ClientProtos.MutateRequest;
37 import org.apache.hadoop.hbase.protobuf.generated
38 .RegionServerStatusProtos.ReportRegionStateTransitionRequest;
39 import org.apache.hadoop.hbase.protobuf.generated.ClientProtos.RegionAction;
40 import org.apache.hadoop.hbase.protobuf.generated.ClientProtos.ScanRequest;
41 import org.apache.hadoop.hbase.protobuf.generated.RPCProtos.RequestHeader;
42 import org.apache.hadoop.hbase.util.ReflectionUtils;
43
44 import com.google.protobuf.Message;
45
46
47
48
49
50
51 @InterfaceAudience.LimitedPrivate({HBaseInterfaceAudience.COPROC, HBaseInterfaceAudience.PHOENIX})
52 @InterfaceStability.Evolving
53 public class RWQueueRpcExecutor extends RpcExecutor {
54 private static final Log LOG = LogFactory.getLog(RWQueueRpcExecutor.class);
55
56 private final List<BlockingQueue<CallRunner>> queues;
57 private final QueueBalancer writeBalancer;
58 private final QueueBalancer readBalancer;
59 private final QueueBalancer scanBalancer;
60 private final int writeHandlersCount;
61 private final int readHandlersCount;
62 private final int scanHandlersCount;
63 private final int numWriteQueues;
64 private final int numReadQueues;
65 private final int numScanQueues;
66
67 public RWQueueRpcExecutor(final String name, final int handlerCount, final int numQueues,
68 final float readShare, final int maxQueueLength,
69 final Configuration conf, final Abortable abortable) {
70 this(name, handlerCount, numQueues, readShare, maxQueueLength, 0,
71 conf, abortable, LinkedBlockingQueue.class);
72 }
73
74 public RWQueueRpcExecutor(final String name, final int handlerCount, final int numQueues,
75 final float readShare, final float scanShare, final int maxQueueLength) {
76 this(name, handlerCount, numQueues, readShare, scanShare, maxQueueLength, null, null);
77 }
78
79 public RWQueueRpcExecutor(final String name, final int handlerCount, final int numQueues,
80 final float readShare, final float scanShare, final int maxQueueLength,
81 final Configuration conf, final Abortable abortable) {
82 this(name, handlerCount, numQueues, readShare, scanShare, maxQueueLength,
83 conf, abortable, LinkedBlockingQueue.class);
84 }
85
86 public RWQueueRpcExecutor(final String name, final int handlerCount, final int numQueues,
87 final float readShare, final int maxQueueLength,
88 final Configuration conf, final Abortable abortable,
89 final Class<? extends BlockingQueue> readQueueClass, Object... readQueueInitArgs) {
90 this(name, handlerCount, numQueues, readShare, 0, maxQueueLength, conf, abortable,
91 readQueueClass, readQueueInitArgs);
92 }
93
94 public RWQueueRpcExecutor(final String name, final int handlerCount, final int numQueues,
95 final float readShare, final float scanShare, final int maxQueueLength,
96 final Configuration conf, final Abortable abortable,
97 final Class<? extends BlockingQueue> readQueueClass, Object... readQueueInitArgs) {
98 this(name, calcNumWriters(handlerCount, readShare), calcNumReaders(handlerCount, readShare),
99 calcNumWriters(numQueues, readShare), calcNumReaders(numQueues, readShare), scanShare,
100 LinkedBlockingQueue.class, new Object[] {maxQueueLength},
101 readQueueClass, ArrayUtils.addAll(new Object[] {maxQueueLength}, readQueueInitArgs));
102 }
103
104 public RWQueueRpcExecutor(final String name, final int writeHandlers, final int readHandlers,
105 final int numWriteQueues, final int numReadQueues,
106 final Class<? extends BlockingQueue> writeQueueClass, Object[] writeQueueInitArgs,
107 final Class<? extends BlockingQueue> readQueueClass, Object[] readQueueInitArgs) {
108 this(name, writeHandlers, readHandlers, numWriteQueues, numReadQueues, 0,
109 writeQueueClass, writeQueueInitArgs, readQueueClass, readQueueInitArgs);
110 }
111
112 public RWQueueRpcExecutor(final String name, int writeHandlers, int readHandlers,
113 int numWriteQueues, int numReadQueues, float scanShare,
114 final Class<? extends BlockingQueue> writeQueueClass, Object[] writeQueueInitArgs,
115 final Class<? extends BlockingQueue> readQueueClass, Object[] readQueueInitArgs) {
116 super(name, Math.max(writeHandlers, numWriteQueues) + Math.max(readHandlers, numReadQueues));
117
118 int numScanQueues = Math.max(0, (int)Math.floor(numReadQueues * scanShare));
119 int scanHandlers = Math.max(0, (int)Math.floor(readHandlers * scanShare));
120 if ((numReadQueues - numScanQueues) > 0) {
121 numReadQueues -= numScanQueues;
122 readHandlers -= scanHandlers;
123 } else {
124 numScanQueues = 0;
125 scanHandlers = 0;
126 }
127
128 this.writeHandlersCount = Math.max(writeHandlers, numWriteQueues);
129 this.readHandlersCount = Math.max(readHandlers, numReadQueues);
130 this.scanHandlersCount = Math.max(scanHandlers, numScanQueues);
131 this.numWriteQueues = numWriteQueues;
132 this.numReadQueues = numReadQueues;
133 this.numScanQueues = numScanQueues;
134 this.writeBalancer = getBalancer(numWriteQueues);
135 this.readBalancer = getBalancer(numReadQueues);
136 this.scanBalancer = getBalancer(numScanQueues);
137
138 queues = new ArrayList<BlockingQueue<CallRunner>>(writeHandlersCount + readHandlersCount);
139 LOG.debug(name + " writeQueues=" + numWriteQueues + " writeHandlers=" + writeHandlersCount +
140 " readQueues=" + numReadQueues + " readHandlers=" + readHandlersCount +
141 ((numScanQueues == 0) ? "" : " scanQueues=" + numScanQueues +
142 " scanHandlers=" + scanHandlersCount));
143
144 for (int i = 0; i < numWriteQueues; ++i) {
145 queues.add((BlockingQueue<CallRunner>)
146 ReflectionUtils.newInstance(writeQueueClass, writeQueueInitArgs));
147 }
148
149 for (int i = 0; i < (numReadQueues + numScanQueues); ++i) {
150 queues.add((BlockingQueue<CallRunner>)
151 ReflectionUtils.newInstance(readQueueClass, readQueueInitArgs));
152 }
153 }
154
155 @Override
156 protected void startHandlers(final int port) {
157 startHandlers(".write", writeHandlersCount, queues, 0, numWriteQueues, port);
158 startHandlers(".read", readHandlersCount, queues, numWriteQueues, numReadQueues, port);
159 startHandlers(".scan", scanHandlersCount, queues,
160 numWriteQueues + numReadQueues, numScanQueues, port);
161 }
162
163 @Override
164 public void dispatch(final CallRunner callTask) throws InterruptedException {
165 RpcServer.Call call = callTask.getCall();
166 int queueIndex;
167 if (isWriteRequest(call.getHeader(), call.param)) {
168 queueIndex = writeBalancer.getNextQueue();
169 } else if (numScanQueues > 0 && isScanRequest(call.getHeader(), call.param)) {
170 queueIndex = numWriteQueues + numReadQueues + scanBalancer.getNextQueue();
171 } else {
172 queueIndex = numWriteQueues + readBalancer.getNextQueue();
173 }
174 queues.get(queueIndex).put(callTask);
175 }
176
177 private boolean isWriteRequest(final RequestHeader header, final Message param) {
178
179 if (param instanceof MultiRequest) {
180 MultiRequest multi = (MultiRequest)param;
181 for (RegionAction regionAction : multi.getRegionActionList()) {
182 for (Action action: regionAction.getActionList()) {
183 if (action.hasMutation()) {
184 return true;
185 }
186 }
187 }
188 }
189 if (param instanceof MutateRequest) {
190 return true;
191 }
192 if (param instanceof ReportRegionStateTransitionRequest) {
193 return true;
194 }
195 return false;
196 }
197
198 private boolean isScanRequest(final RequestHeader header, final Message param) {
199 if (param instanceof ScanRequest) {
200
201 ScanRequest request = (ScanRequest)param;
202 return request.hasScannerId();
203 }
204 return false;
205 }
206
207 @Override
208 public int getQueueLength() {
209 int length = 0;
210 for (final BlockingQueue<CallRunner> queue: queues) {
211 length += queue.size();
212 }
213 return length;
214 }
215
216 @Override
217 protected List<BlockingQueue<CallRunner>> getQueues() {
218 return queues;
219 }
220
221
222
223
224
225 private static int calcNumWriters(final int count, final float readShare) {
226 return Math.max(1, count - Math.max(1, (int)Math.round(count * readShare)));
227 }
228
229
230
231
232
233 private static int calcNumReaders(final int count, final float readShare) {
234 return count - calcNumWriters(count, readShare);
235 }
236 }