View Javadoc

1   /**
2    * Copyright The Apache Software Foundation
3    *
4    * Licensed to the Apache Software Foundation (ASF) under one
5    * or more contributor license agreements.  See the NOTICE file
6    * distributed with this work for additional information
7    * regarding copyright ownership.  The ASF licenses this file
8    * to you under the Apache License, Version 2.0 (the
9    * "License"); you may not use this file except in compliance
10   * with the License.  You may obtain a copy of the License at
11   *
12   *     http://www.apache.org/licenses/LICENSE-2.0
13   *
14   * Unless required by applicable law or agreed to in writing, software
15   * distributed under the License is distributed on an "AS IS" BASIS,
16   * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
17   * See the License for the specific language governing permissions and
18   * limitations under the License.
19   */
20  
21  package org.apache.hadoop.hbase;
22  
23  import org.apache.hadoop.hbase.classification.InterfaceAudience;
24  import org.apache.hadoop.hbase.classification.InterfaceStability;
25  import org.apache.hadoop.hbase.protobuf.ProtobufUtil;
26  import org.apache.hadoop.hbase.protobuf.generated.ClusterStatusProtos;
27  import org.apache.hadoop.hbase.protobuf.generated.HBaseProtos.Coprocessor;
28  import org.apache.hadoop.hbase.replication.ReplicationLoadSink;
29  import org.apache.hadoop.hbase.replication.ReplicationLoadSource;
30  import org.apache.hadoop.hbase.util.Bytes;
31  import org.apache.hadoop.hbase.util.Strings;
32  
33  import java.util.Arrays;
34  import java.util.List;
35  import java.util.Map;
36  import java.util.TreeMap;
37  import java.util.TreeSet;
38  
39  /**
40   * This class is used for exporting current state of load on a RegionServer.
41   */
42  @InterfaceAudience.Public
43  @InterfaceStability.Evolving
44  public class ServerLoad {
45    private int stores = 0;
46    private int storefiles = 0;
47    private int storeUncompressedSizeMB = 0;
48    private int storefileSizeMB = 0;
49    private int memstoreSizeMB = 0;
50    private int storefileIndexSizeMB = 0;
51    private long readRequestsCount = 0;
52    private long writeRequestsCount = 0;
53    private int rootIndexSizeKB = 0;
54    private int totalStaticIndexSizeKB = 0;
55    private int totalStaticBloomSizeKB = 0;
56    private long totalCompactingKVs = 0;
57    private long currentCompactedKVs = 0;
58  
59    public ServerLoad(ClusterStatusProtos.ServerLoad serverLoad) {
60      this.serverLoad = serverLoad;
61      for (ClusterStatusProtos.RegionLoad rl: serverLoad.getRegionLoadsList()) {
62        stores += rl.getStores();
63        storefiles += rl.getStorefiles();
64        storeUncompressedSizeMB += rl.getStoreUncompressedSizeMB();
65        storefileSizeMB += rl.getStorefileSizeMB();
66        memstoreSizeMB += rl.getMemstoreSizeMB();
67        storefileIndexSizeMB += rl.getStorefileIndexSizeMB();
68        readRequestsCount += rl.getReadRequestsCount();
69        writeRequestsCount += rl.getWriteRequestsCount();
70        rootIndexSizeKB += rl.getRootIndexSizeKB();
71        totalStaticIndexSizeKB += rl.getTotalStaticIndexSizeKB();
72        totalStaticBloomSizeKB += rl.getTotalStaticBloomSizeKB();
73        totalCompactingKVs += rl.getTotalCompactingKVs();
74        currentCompactedKVs += rl.getCurrentCompactedKVs();
75      }
76  
77    }
78  
79    // NOTE: Function name cannot start with "get" because then an OpenDataException is thrown because
80    // HBaseProtos.ServerLoad cannot be converted to an open data type(see HBASE-5967).
81    /* @return the underlying ServerLoad protobuf object */
82    public ClusterStatusProtos.ServerLoad obtainServerLoadPB() {
83      return serverLoad;
84    }
85  
86    protected ClusterStatusProtos.ServerLoad serverLoad;
87  
88    /* @return number of requests  since last report. */
89    public long getNumberOfRequests() {
90      return serverLoad.getNumberOfRequests();
91    }
92    public boolean hasNumberOfRequests() {
93      return serverLoad.hasNumberOfRequests();
94    }
95  
96    /* @return total Number of requests from the start of the region server. */
97    public long getTotalNumberOfRequests() {
98      return serverLoad.getTotalNumberOfRequests();
99    }
100   public boolean hasTotalNumberOfRequests() {
101     return serverLoad.hasTotalNumberOfRequests();
102   }
103 
104   /* @return the amount of used heap, in MB. */
105   public int getUsedHeapMB() {
106     return serverLoad.getUsedHeapMB();
107   }
108   public boolean hasUsedHeapMB() {
109     return serverLoad.hasUsedHeapMB();
110   }
111 
112   /* @return the maximum allowable size of the heap, in MB. */
113   public int getMaxHeapMB() {
114     return serverLoad.getMaxHeapMB();
115   }
116   public boolean hasMaxHeapMB() {
117     return serverLoad.hasMaxHeapMB();
118   }
119 
120   public int getStores() {
121     return stores;
122   }
123 
124   public int getStorefiles() {
125     return storefiles;
126   }
127 
128   public int getStoreUncompressedSizeMB() {
129     return storeUncompressedSizeMB;
130   }
131 
132   public int getStorefileSizeInMB() {
133     return storefileSizeMB;
134   }
135 
136   public int getMemstoreSizeInMB() {
137     return memstoreSizeMB;
138   }
139 
140   public int getStorefileIndexSizeInMB() {
141     return storefileIndexSizeMB;
142   }
143 
144   public long getReadRequestsCount() {
145     return readRequestsCount;
146   }
147 
148   public long getWriteRequestsCount() {
149     return writeRequestsCount;
150   }
151 
152   public int getRootIndexSizeKB() {
153     return rootIndexSizeKB;
154   }
155 
156   public int getTotalStaticIndexSizeKB() {
157     return totalStaticIndexSizeKB;
158   }
159 
160   public int getTotalStaticBloomSizeKB() {
161     return totalStaticBloomSizeKB;
162   }
163 
164   public long getTotalCompactingKVs() {
165     return totalCompactingKVs;
166   }
167 
168   public long getCurrentCompactedKVs() {
169     return currentCompactedKVs;
170   }
171 
172   /**
173    * @return the number of regions
174    */
175   public int getNumberOfRegions() {
176     return serverLoad.getRegionLoadsCount();
177   }
178 
179   public int getInfoServerPort() {
180     return serverLoad.getInfoServerPort();
181   }
182 
183   /**
184    * Call directly from client such as hbase shell
185    * @return the list of ReplicationLoadSource
186    */
187   public List<ReplicationLoadSource> getReplicationLoadSourceList() {
188     return ProtobufUtil.toReplicationLoadSourceList(serverLoad.getReplLoadSourceList());
189   }
190 
191   /**
192    * Call directly from client such as hbase shell
193    * @return ReplicationLoadSink
194    */
195   public ReplicationLoadSink getReplicationLoadSink() {
196     if (serverLoad.hasReplLoadSink()) {
197       return ProtobufUtil.toReplicationLoadSink(serverLoad.getReplLoadSink());
198     } else {
199       return null;
200     }
201   }
202 
203   /**
204    * Originally, this method factored in the effect of requests going to the
205    * server as well. However, this does not interact very well with the current
206    * region rebalancing code, which only factors number of regions. For the
207    * interim, until we can figure out how to make rebalancing use all the info
208    * available, we're just going to make load purely the number of regions.
209    *
210    * @return load factor for this server
211    */
212   public int getLoad() {
213     // See above comment
214     // int load = numberOfRequests == 0 ? 1 : numberOfRequests;
215     // load *= numberOfRegions == 0 ? 1 : numberOfRegions;
216     // return load;
217     return getNumberOfRegions();
218   }
219 
220   /**
221    * @return region load metrics
222    */
223   public Map<byte[], RegionLoad> getRegionsLoad() {
224     Map<byte[], RegionLoad> regionLoads =
225       new TreeMap<byte[], RegionLoad>(Bytes.BYTES_COMPARATOR);
226     for (ClusterStatusProtos.RegionLoad rl : serverLoad.getRegionLoadsList()) {
227       RegionLoad regionLoad = new RegionLoad(rl);
228       regionLoads.put(regionLoad.getName(), regionLoad);
229     }
230     return regionLoads;
231   }
232 
233   /**
234    * Return the RegionServer-level coprocessors
235    * @return string array of loaded RegionServer-level coprocessors
236    */
237   public String[] getRegionServerCoprocessors() {
238     List<Coprocessor> list = obtainServerLoadPB().getCoprocessorsList();
239     String [] ret = new String[list.size()];
240     int i = 0;
241     for (Coprocessor elem : list) {
242       ret[i++] = elem.getName();
243     }
244 
245     return ret;
246   }
247 
248   /**
249    * Return the RegionServer-level and Region-level coprocessors
250    * @return string array of loaded RegionServer-level and
251    *         Region-level coprocessors
252    */
253   public String[] getRsCoprocessors() {
254     // Need a set to remove duplicates, but since generated Coprocessor class
255     // is not Comparable, make it a Set<String> instead of Set<Coprocessor>
256     TreeSet<String> coprocessSet = new TreeSet<String>();
257     for (Coprocessor coprocessor : obtainServerLoadPB().getCoprocessorsList()) {
258       coprocessSet.add(coprocessor.getName());
259     }
260     return coprocessSet.toArray(new String[coprocessSet.size()]);
261   }
262 
263   /**
264    * @return number of requests per second received since the last report
265    */
266   public double getRequestsPerSecond() {
267     return getNumberOfRequests();
268   }
269 
270   /**
271    * @see java.lang.Object#toString()
272    */
273   @Override
274   public String toString() {
275      StringBuilder sb =
276         Strings.appendKeyValue(new StringBuilder(), "requestsPerSecond",
277           Double.valueOf(getRequestsPerSecond()));
278     Strings.appendKeyValue(sb, "numberOfOnlineRegions", Integer.valueOf(getNumberOfRegions()));
279     sb = Strings.appendKeyValue(sb, "usedHeapMB", Integer.valueOf(this.getUsedHeapMB()));
280     sb = Strings.appendKeyValue(sb, "maxHeapMB", Integer.valueOf(getMaxHeapMB()));
281     sb = Strings.appendKeyValue(sb, "numberOfStores", Integer.valueOf(this.stores));
282     sb = Strings.appendKeyValue(sb, "numberOfStorefiles", Integer.valueOf(this.storefiles));
283     sb =
284         Strings.appendKeyValue(sb, "storefileUncompressedSizeMB",
285           Integer.valueOf(this.storeUncompressedSizeMB));
286     sb = Strings.appendKeyValue(sb, "storefileSizeMB", Integer.valueOf(this.storefileSizeMB));
287     if (this.storeUncompressedSizeMB != 0) {
288       sb =
289           Strings.appendKeyValue(
290             sb,
291             "compressionRatio",
292             String.format("%.4f", (float) this.storefileSizeMB
293                 / (float) this.storeUncompressedSizeMB));
294     }
295     sb = Strings.appendKeyValue(sb, "memstoreSizeMB", Integer.valueOf(this.memstoreSizeMB));
296     sb =
297         Strings.appendKeyValue(sb, "storefileIndexSizeMB",
298           Integer.valueOf(this.storefileIndexSizeMB));
299     sb = Strings.appendKeyValue(sb, "readRequestsCount", Long.valueOf(this.readRequestsCount));
300     sb = Strings.appendKeyValue(sb, "writeRequestsCount", Long.valueOf(this.writeRequestsCount));
301     sb = Strings.appendKeyValue(sb, "rootIndexSizeKB", Integer.valueOf(this.rootIndexSizeKB));
302     sb =
303         Strings.appendKeyValue(sb, "totalStaticIndexSizeKB",
304           Integer.valueOf(this.totalStaticIndexSizeKB));
305     sb =
306         Strings.appendKeyValue(sb, "totalStaticBloomSizeKB",
307           Integer.valueOf(this.totalStaticBloomSizeKB));
308     sb = Strings.appendKeyValue(sb, "totalCompactingKVs", Long.valueOf(this.totalCompactingKVs));
309     sb = Strings.appendKeyValue(sb, "currentCompactedKVs", Long.valueOf(this.currentCompactedKVs));
310     float compactionProgressPct = Float.NaN;
311     if (this.totalCompactingKVs > 0) {
312       compactionProgressPct =
313           Float.valueOf((float) this.currentCompactedKVs / this.totalCompactingKVs);
314     }
315     sb = Strings.appendKeyValue(sb, "compactionProgressPct", compactionProgressPct);
316 
317     String[] coprocessorStrings = getRsCoprocessors();
318     if (coprocessorStrings != null) {
319       sb = Strings.appendKeyValue(sb, "coprocessors", Arrays.toString(coprocessorStrings));
320     }
321     return sb.toString();
322   }
323 
324   public static final ServerLoad EMPTY_SERVERLOAD =
325     new ServerLoad(ClusterStatusProtos.ServerLoad.newBuilder().build());
326 }