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.hdfs.util;
019
020import java.util.HashMap;
021import java.util.LinkedList;
022import java.util.Map;
023import java.util.Queue;
024
025import org.apache.commons.logging.Log;
026import org.apache.commons.logging.LogFactory;
027import org.apache.hadoop.HadoopIllegalArgumentException;
028import org.apache.hadoop.classification.InterfaceAudience;
029import org.apache.hadoop.util.Time;
030
031import com.google.common.base.Preconditions;
032
033/**
034 * Manage byte array creation and release. 
035 */
036@InterfaceAudience.Private
037public abstract class ByteArrayManager {
038  static final Log LOG = LogFactory.getLog(ByteArrayManager.class);
039  private static final ThreadLocal<StringBuilder> debugMessage = new ThreadLocal<StringBuilder>() {
040    protected StringBuilder initialValue() {
041      return new StringBuilder();
042    }
043  };
044
045  private static void logDebugMessage() {
046    final StringBuilder b = debugMessage.get();
047    LOG.debug(b);
048    b.setLength(0);
049  }
050
051  static final int MIN_ARRAY_LENGTH = 32;
052  static final byte[] EMPTY_BYTE_ARRAY = {};
053
054  /**
055   * @return the least power of two greater than or equal to n, i.e. return
056   *         the least integer x with x >= n and x a power of two.
057   *
058   * @throws HadoopIllegalArgumentException
059   *           if n <= 0.
060   */
061  public static int leastPowerOfTwo(final int n) {
062    if (n <= 0) {
063      throw new HadoopIllegalArgumentException("n = " + n + " <= 0");
064    }
065
066    final int highestOne = Integer.highestOneBit(n);
067    if (highestOne == n) {
068      return n; // n is a power of two.
069    }
070    final int roundUp = highestOne << 1;
071    if (roundUp < 0) {
072      final long overflow = ((long) highestOne) << 1;
073      throw new ArithmeticException(
074          "Overflow: for n = " + n + ", the least power of two (the least"
075          + " integer x with x >= n and x a power of two) = "
076          + overflow + " > Integer.MAX_VALUE = " + Integer.MAX_VALUE);
077    }
078    return roundUp;
079  }
080
081  /**
082   * A counter with a time stamp so that it is reset automatically
083   * if there is no increment for the time period.
084   */
085  static class Counter {
086    private final long countResetTimePeriodMs;
087    private long count = 0L;
088    private long timestamp = Time.monotonicNow();
089
090    Counter(long countResetTimePeriodMs) {
091      this.countResetTimePeriodMs = countResetTimePeriodMs;
092    }
093
094    synchronized long getCount() {
095      return count;
096    }
097
098    /**
099     * Increment the counter, and reset it if there is no increment
100     * for acertain time period.
101     *
102     * @return the new count.
103     */
104    synchronized long increment() {
105      final long now = Time.monotonicNow();
106      if (now - timestamp > countResetTimePeriodMs) {
107        count = 0; // reset the counter
108      }
109      timestamp = now;
110      return ++count;
111    }
112  }
113
114  /** A map from integers to counters. */
115  static class CounterMap {
116    /** @see ByteArrayManager.Conf#countResetTimePeriodMs */
117    private final long countResetTimePeriodMs;
118    private final Map<Integer, Counter> map = new HashMap<Integer, Counter>();
119
120    private CounterMap(long countResetTimePeriodMs) {
121      this.countResetTimePeriodMs = countResetTimePeriodMs;
122    }
123
124    /**
125     * @return the counter for the given key;
126     *         and create a new counter if it does not exist.
127     */
128    synchronized Counter get(final Integer key, final boolean createIfNotExist) {
129      Counter count = map.get(key);
130      if (count == null && createIfNotExist) {
131        count = new Counter(countResetTimePeriodMs);
132        map.put(key, count);
133      }
134      return count;
135    }
136
137    synchronized void clear() {
138      map.clear();
139    }
140  }
141
142  /** Manage byte arrays with the same fixed length. */
143  static class FixedLengthManager {
144    private final int byteArrayLength;
145    private final int maxAllocated;
146    private final Queue<byte[]> freeQueue = new LinkedList<byte[]>();
147
148    private int numAllocated = 0;
149
150    FixedLengthManager(int arrayLength, int maxAllocated) {
151      this.byteArrayLength = arrayLength;
152      this.maxAllocated = maxAllocated;
153    }
154
155    /**
156     * Allocate a byte array.
157     *
158     * If the number of allocated arrays >= maximum, the current thread is
159     * blocked until the number of allocated arrays drops to below the maximum.
160     * 
161     * The byte array allocated by this method must be returned for recycling
162     * via the {@link FixedLengthManager#recycle(byte[])} method.
163     */
164    synchronized byte[] allocate() throws InterruptedException {
165      if (LOG.isDebugEnabled()) {
166        debugMessage.get().append(", ").append(this);
167      }
168      for(; numAllocated >= maxAllocated;) {
169        if (LOG.isDebugEnabled()) {
170          debugMessage.get().append(": wait ...");
171          logDebugMessage();
172        }
173
174        wait();
175
176        if (LOG.isDebugEnabled()) {
177          debugMessage.get().append("wake up: ").append(this);
178        }
179      }
180      numAllocated++;
181
182      final byte[] array = freeQueue.poll();
183      if (LOG.isDebugEnabled()) {
184        debugMessage.get().append(", recycled? ").append(array != null);
185      }
186      return array != null? array : new byte[byteArrayLength];
187    }
188
189    /**
190     * Recycle the given byte array, which must have the same length as the
191     * array length managed by this object.
192     *
193     * The byte array may or may not be allocated
194     * by the {@link FixedLengthManager#allocate()} method.
195     */
196    synchronized int recycle(byte[] array) {
197      Preconditions.checkNotNull(array);
198      Preconditions.checkArgument(array.length == byteArrayLength);
199      if (LOG.isDebugEnabled()) {
200        debugMessage.get().append(", ").append(this);
201      }
202
203      notify();
204      numAllocated--;
205      if (numAllocated < 0) {
206        // it is possible to drop below 0 since
207        // some byte arrays may not be created by the allocate() method.
208        numAllocated = 0;
209      }
210
211      if (freeQueue.size() < maxAllocated - numAllocated) {
212        if (LOG.isDebugEnabled()) {
213          debugMessage.get().append(", freeQueue.offer");
214        }
215        freeQueue.offer(array);
216      }
217      return freeQueue.size();
218    }
219
220    @Override
221    public synchronized String toString() {
222      return "[" + byteArrayLength + ": " + numAllocated + "/"
223          + maxAllocated + ", free=" + freeQueue.size() + "]";
224    }
225  }
226
227  /** A map from array lengths to byte array managers. */
228  static class ManagerMap {
229    private final int countLimit;
230    private final Map<Integer, FixedLengthManager> map = new HashMap<Integer, FixedLengthManager>();
231
232    ManagerMap(int countLimit) {
233      this.countLimit = countLimit;
234    }
235
236    /** @return the manager for the given array length. */
237    synchronized FixedLengthManager get(final Integer arrayLength,
238        final boolean createIfNotExist) {
239      FixedLengthManager manager = map.get(arrayLength);
240      if (manager == null && createIfNotExist) {
241        manager = new FixedLengthManager(arrayLength, countLimit);
242        map.put(arrayLength, manager);
243      }
244      return manager;
245    }
246
247    synchronized void clear() {
248      map.clear();
249    }
250  }
251
252  public static class Conf {
253    /**
254     * The count threshold for each array length so that a manager is created
255     * only after the allocation count exceeds the threshold.
256     */
257    private final int countThreshold;
258    /**
259     * The maximum number of arrays allowed for each array length.
260     */
261    private final int countLimit;
262    /**
263     * The time period in milliseconds that the allocation count for each array
264     * length is reset to zero if there is no increment.
265     */
266    private final long countResetTimePeriodMs;
267
268    public Conf(int countThreshold, int countLimit, long countResetTimePeriodMs) {
269      this.countThreshold = countThreshold;
270      this.countLimit = countLimit;
271      this.countResetTimePeriodMs = countResetTimePeriodMs;
272    }
273  }
274
275  /**
276   * Create a byte array for the given length, where the length of
277   * the returned array is larger than or equal to the given length.
278   *
279   * The current thread may be blocked if some resource is unavailable.
280   * 
281   * The byte array created by this method must be released
282   * via the {@link ByteArrayManager#release(byte[])} method.
283   *
284   * @return a byte array with length larger than or equal to the given length.
285   */
286  public abstract byte[] newByteArray(int size) throws InterruptedException;
287  
288  /**
289   * Release the given byte array.
290   * 
291   * The byte array may or may not be created
292   * by the {@link ByteArrayManager#newByteArray(int)} method.
293   * 
294   * @return the number of free array.
295   */
296  public abstract int release(byte[] array);
297
298  public static ByteArrayManager newInstance(Conf conf) {
299    return conf == null? new NewByteArrayWithoutLimit(): new Impl(conf);
300  }
301
302  /**
303   * A dummy implementation which simply calls new byte[].
304   */
305  static class NewByteArrayWithoutLimit extends ByteArrayManager {
306    @Override
307    public byte[] newByteArray(int size) throws InterruptedException {
308      return new byte[size];
309    }
310    
311    @Override
312    public int release(byte[] array) {
313      return 0;
314    }
315  }
316
317  /**
318   * Manage byte array allocation and provide a mechanism for recycling the byte
319   * array objects.
320   */
321  static class Impl extends ByteArrayManager {
322    private final Conf conf;
323  
324    private final CounterMap counters;
325    private final ManagerMap managers;
326  
327    Impl(Conf conf) {
328      this.conf = conf;
329      this.counters = new CounterMap(conf.countResetTimePeriodMs);
330      this.managers = new ManagerMap(conf.countLimit);
331    }
332  
333    /**
334     * Allocate a byte array, where the length of the allocated array
335     * is the least power of two of the given length
336     * unless the given length is less than {@link #MIN_ARRAY_LENGTH}.
337     * In such case, the returned array length is equal to {@link #MIN_ARRAY_LENGTH}.
338     *
339     * If the number of allocated arrays exceeds the capacity,
340     * the current thread is blocked until
341     * the number of allocated arrays drops to below the capacity.
342     * 
343     * The byte array allocated by this method must be returned for recycling
344     * via the {@link Impl#release(byte[])} method.
345     *
346     * @return a byte array with length larger than or equal to the given length.
347     */
348    @Override
349    public byte[] newByteArray(final int arrayLength) throws InterruptedException {
350      Preconditions.checkArgument(arrayLength >= 0);
351      if (LOG.isDebugEnabled()) {
352        debugMessage.get().append("allocate(").append(arrayLength).append(")");
353      }
354  
355      final byte[] array;
356      if (arrayLength == 0) {
357        array = EMPTY_BYTE_ARRAY;
358      } else {
359        final int powerOfTwo = arrayLength <= MIN_ARRAY_LENGTH?
360            MIN_ARRAY_LENGTH: leastPowerOfTwo(arrayLength);
361        final long count = counters.get(powerOfTwo, true).increment();
362        final boolean aboveThreshold = count > conf.countThreshold;
363        // create a new manager only if the count is above threshold.
364        final FixedLengthManager manager = managers.get(powerOfTwo, aboveThreshold);
365  
366        if (LOG.isDebugEnabled()) {
367          debugMessage.get().append(": count=").append(count)
368              .append(aboveThreshold? ", aboveThreshold": ", belowThreshold");
369        }
370        array = manager != null? manager.allocate(): new byte[powerOfTwo];
371      }
372  
373      if (LOG.isDebugEnabled()) {
374        debugMessage.get().append(", return byte[").append(array.length).append("]");
375        logDebugMessage();
376      }
377      return array;
378    }
379  
380    /**
381     * Recycle the given byte array.
382     * 
383     * The byte array may or may not be allocated
384     * by the {@link Impl#newByteArray(int)} method.
385     * 
386     * This is a non-blocking call.
387     */
388    @Override
389    public int release(final byte[] array) {
390      Preconditions.checkNotNull(array);
391      if (LOG.isDebugEnabled()) {
392        debugMessage.get().append("recycle: array.length=").append(array.length);
393      }
394  
395      final int freeQueueSize;
396      if (array.length == 0) {
397        freeQueueSize = -1;
398      } else {
399        final FixedLengthManager manager = managers.get(array.length, false);
400        freeQueueSize = manager == null? -1: manager.recycle(array);
401      }
402  
403      if (LOG.isDebugEnabled()) {
404        debugMessage.get().append(", freeQueueSize=").append(freeQueueSize);
405        logDebugMessage();
406      }
407      return freeQueueSize;
408    }
409  
410    CounterMap getCounters() {
411      return counters;
412    }
413  
414    ManagerMap getManagers() {
415      return managers;
416    }
417  }
418}