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.util;
019
020import java.io.PrintStream;
021import java.util.AbstractCollection;
022import java.util.Arrays;
023import java.util.Collection;
024import java.util.ConcurrentModificationException;
025import java.util.Iterator;
026
027import org.apache.hadoop.HadoopIllegalArgumentException;
028import org.apache.hadoop.classification.InterfaceAudience;
029
030import com.google.common.annotations.VisibleForTesting;
031
032/**
033 * A low memory footprint {@link GSet} implementation,
034 * which uses an array for storing the elements
035 * and linked lists for collision resolution.
036 *
037 * No rehash will be performed.
038 * Therefore, the internal array will never be resized.
039 *
040 * This class does not support null element.
041 *
042 * This class is not thread safe.
043 *
044 * @param <K> Key type for looking up the elements
045 * @param <E> Element type, which must be
046 *       (1) a subclass of K, and
047 *       (2) implementing {@link LinkedElement} interface.
048 */
049@InterfaceAudience.Private
050public class LightWeightGSet<K, E extends K> implements GSet<K, E> {
051  /**
052   * Elements of {@link LightWeightGSet}.
053   */
054  public interface LinkedElement {
055    /** Set the next element. */
056    void setNext(LinkedElement next);
057
058    /** Get the next element. */
059    LinkedElement getNext();
060  }
061
062  static final int MAX_ARRAY_LENGTH = 1 << 30; //prevent int overflow problem
063  static final int MIN_ARRAY_LENGTH = 1;
064
065  /**
066   * An internal array of entries, which are the rows of the hash table.
067   * The size must be a power of two.
068   */
069  protected LinkedElement[] entries;
070  /** A mask for computing the array index from the hash value of an element. */
071  protected int hash_mask;
072  /** The size of the set (not the entry array). */
073  protected int size = 0;
074  /** Modification version for fail-fast.
075   * @see ConcurrentModificationException
076   */
077  protected int modification = 0;
078
079  private Collection<E> values;
080
081  protected LightWeightGSet() {
082  }
083
084  /**
085   * @param recommended_length Recommended size of the internal array.
086   */
087  public LightWeightGSet(final int recommended_length) {
088    final int actual = actualArrayLength(recommended_length);
089    if (LOG.isDebugEnabled()) {
090      LOG.debug("recommended=" + recommended_length + ", actual=" + actual);
091    }
092    entries = new LinkedElement[actual];
093    hash_mask = entries.length - 1;
094  }
095
096  //compute actual length
097  protected static int actualArrayLength(int recommended) {
098    if (recommended > MAX_ARRAY_LENGTH) {
099      return MAX_ARRAY_LENGTH;
100    } else if (recommended < MIN_ARRAY_LENGTH) {
101      return MIN_ARRAY_LENGTH;
102    } else {
103      final int a = Integer.highestOneBit(recommended);
104      return a == recommended? a: a << 1;
105    }
106  }
107
108  @Override
109  public int size() {
110    return size;
111  }
112
113  protected int getIndex(final K key) {
114    return key.hashCode() & hash_mask;
115  }
116
117  protected E convert(final LinkedElement e){
118    @SuppressWarnings("unchecked")
119    final E r = (E)e;
120    return r;
121  }
122
123  @Override
124  public E get(final K key) {
125    //validate key
126    if (key == null) {
127      throw new NullPointerException("key == null");
128    }
129
130    //find element
131    final int index = getIndex(key);
132    for(LinkedElement e = entries[index]; e != null; e = e.getNext()) {
133      if (e.equals(key)) {
134        return convert(e);
135      }
136    }
137    //element not found
138    return null;
139  }
140
141  @Override
142  public boolean contains(final K key) {
143    return get(key) != null;
144  }
145
146  @Override
147  public E put(final E element) {
148    // validate element
149    if (element == null) {
150      throw new NullPointerException("Null element is not supported.");
151    }
152    LinkedElement e = null;
153    try {
154      e = (LinkedElement)element;
155    } catch (ClassCastException ex) {
156      throw new HadoopIllegalArgumentException(
157          "!(element instanceof LinkedElement), element.getClass()="
158          + element.getClass());
159    }
160
161    // find index
162    final int index = getIndex(element);
163
164    // remove if it already exists
165    final E existing = remove(index, element);
166
167    // insert the element to the head of the linked list
168    modification++;
169    size++;
170    e.setNext(entries[index]);
171    entries[index] = e;
172
173    return existing;
174  }
175
176  /**
177   * Remove the element corresponding to the key,
178   * given key.hashCode() == index.
179   *
180   * @return If such element exists, return it.
181   *         Otherwise, return null.
182   */
183  protected E remove(final int index, final K key) {
184    if (entries[index] == null) {
185      return null;
186    } else if (entries[index].equals(key)) {
187      //remove the head of the linked list
188      modification++;
189      size--;
190      final LinkedElement e = entries[index];
191      entries[index] = e.getNext();
192      e.setNext(null);
193      return convert(e);
194    } else {
195      //head != null and key is not equal to head
196      //search the element
197      LinkedElement prev = entries[index];
198      for(LinkedElement curr = prev.getNext(); curr != null; ) {
199        if (curr.equals(key)) {
200          //found the element, remove it
201          modification++;
202          size--;
203          prev.setNext(curr.getNext());
204          curr.setNext(null);
205          return convert(curr);
206        } else {
207          prev = curr;
208          curr = curr.getNext();
209        }
210      }
211      //element not found
212      return null;
213    }
214  }
215
216  @Override
217  public E remove(final K key) {
218    //validate key
219    if (key == null) {
220      throw new NullPointerException("key == null");
221    }
222    return remove(getIndex(key), key);
223  }
224
225  @Override
226  public Collection<E> values() {
227    if (values == null) {
228      values = new Values();
229    }
230    return values;
231  }
232
233  private final class Values extends AbstractCollection<E> {
234
235    @Override
236    public Iterator<E> iterator() {
237      return LightWeightGSet.this.iterator();
238    }
239
240    @Override
241    public int size() {
242      return size;
243    }
244
245    @SuppressWarnings("unchecked")
246    @Override
247    public boolean contains(Object o) {
248      return LightWeightGSet.this.contains((K)o);
249    }
250
251    @Override
252    public void clear() {
253      LightWeightGSet.this.clear();
254    }
255  }
256
257  @Override
258  public Iterator<E> iterator() {
259    return new SetIterator();
260  }
261
262  @Override
263  public String toString() {
264    final StringBuilder b = new StringBuilder(getClass().getSimpleName());
265    b.append("(size=").append(size)
266     .append(String.format(", %08x", hash_mask))
267     .append(", modification=").append(modification)
268     .append(", entries.length=").append(entries.length)
269     .append(")");
270    return b.toString();
271  }
272
273  /** Print detailed information of this object. */
274  public void printDetails(final PrintStream out) {
275    out.print(this + ", entries = [");
276    for(int i = 0; i < entries.length; i++) {
277      if (entries[i] != null) {
278        LinkedElement e = entries[i];
279        out.print("\n  " + i + ": " + e);
280        for(e = e.getNext(); e != null; e = e.getNext()) {
281          out.print(" -> " + e);
282        }
283      }
284    }
285    out.println("\n]");
286  }
287
288  public class SetIterator implements Iterator<E> {
289    /** The starting modification for fail-fast. */
290    private int iterModification = modification;
291    /** The current index of the entry array. */
292    private int index = -1;
293    private LinkedElement cur = null;
294    private LinkedElement next = nextNonemptyEntry();
295    private boolean trackModification = true;
296
297    /** Find the next nonempty entry starting at (index + 1). */
298    private LinkedElement nextNonemptyEntry() {
299      for(index++; index < entries.length && entries[index] == null; index++);
300      return index < entries.length? entries[index]: null;
301    }
302
303    private void ensureNext() {
304      if (trackModification && modification != iterModification) {
305        throw new ConcurrentModificationException("modification=" + modification
306            + " != iterModification = " + iterModification);
307      }
308      if (next != null) {
309        return;
310      }
311      if (cur == null) {
312        return;
313      }
314      next = cur.getNext();
315      if (next == null) {
316        next = nextNonemptyEntry();
317      }
318    }
319
320    @Override
321    public boolean hasNext() {
322      ensureNext();
323      return next != null;
324    }
325
326    @Override
327    public E next() {
328      ensureNext();
329      if (next == null) {
330        throw new IllegalStateException("There are no more elements");
331      }
332      cur = next;
333      next = null;
334      return convert(cur);
335    }
336
337    @SuppressWarnings("unchecked")
338    @Override
339    public void remove() {
340      ensureNext();
341      if (cur == null) {
342        throw new IllegalStateException("There is no current element " +
343            "to remove");
344      }
345      LightWeightGSet.this.remove((K)cur);
346      iterModification++;
347      cur = null;
348    }
349
350    public void setTrackModification(boolean trackModification) {
351      this.trackModification = trackModification;
352    }
353  }
354  
355  /**
356   * Let t = percentage of max memory.
357   * Let e = round(log_2 t).
358   * Then, we choose capacity = 2^e/(size of reference),
359   * unless it is outside the close interval [1, 2^30].
360   */
361  public static int computeCapacity(double percentage, String mapName) {
362    return computeCapacity(Runtime.getRuntime().maxMemory(), percentage,
363        mapName);
364  }
365  
366  @VisibleForTesting
367  static int computeCapacity(long maxMemory, double percentage,
368      String mapName) {
369    if (percentage > 100.0 || percentage < 0.0) {
370      throw new HadoopIllegalArgumentException("Percentage " + percentage
371          + " must be greater than or equal to 0 "
372          + " and less than or equal to 100");
373    }
374    if (maxMemory < 0) {
375      throw new HadoopIllegalArgumentException("Memory " + maxMemory
376          + " must be greater than or equal to 0");
377    }
378    if (percentage == 0.0 || maxMemory == 0) {
379      return 0;
380    }
381    //VM detection
382    //See http://java.sun.com/docs/hotspot/HotSpotFAQ.html#64bit_detection
383    final String vmBit = System.getProperty("sun.arch.data.model");
384
385    //Percentage of max memory
386    final double percentDivisor = 100.0/percentage;
387    final double percentMemory = maxMemory/percentDivisor;
388    
389    //compute capacity
390    final int e1 = (int)(Math.log(percentMemory)/Math.log(2.0) + 0.5);
391    final int e2 = e1 - ("32".equals(vmBit)? 2: 3);
392    final int exponent = e2 < 0? 0: e2 > 30? 30: e2;
393    final int c = 1 << exponent;
394
395    LOG.info("Computing capacity for map " + mapName);
396    LOG.info("VM type       = " + vmBit + "-bit");
397    LOG.info(percentage + "% max memory "
398        + StringUtils.TraditionalBinaryPrefix.long2String(maxMemory, "B", 1)
399        + " = "
400        + StringUtils.TraditionalBinaryPrefix.long2String((long) percentMemory,
401            "B", 1));
402    LOG.info("capacity      = 2^" + exponent + " = " + c + " entries");
403    return c;
404  }
405  
406  public void clear() {
407    modification++;
408    Arrays.fill(entries, null);
409    size = 0;
410  }
411}