You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@hbase.apache.org by ap...@apache.org on 2015/04/30 23:22:05 UTC

[5/6] hbase git commit: HBASE-13420 RegionEnvironment.offerExecutionLatency Blocks Threads under Heavy Load

HBASE-13420 RegionEnvironment.offerExecutionLatency Blocks Threads under Heavy Load


Project: http://git-wip-us.apache.org/repos/asf/hbase/repo
Commit: http://git-wip-us.apache.org/repos/asf/hbase/commit/786a413e
Tree: http://git-wip-us.apache.org/repos/asf/hbase/tree/786a413e
Diff: http://git-wip-us.apache.org/repos/asf/hbase/diff/786a413e

Branch: refs/heads/branch-1.1
Commit: 786a413ec9c7b29311a49d0afb972fad7b4d3340
Parents: 94a29e2
Author: Andrew Purtell <ap...@apache.org>
Authored: Thu Apr 30 12:54:55 2015 -0700
Committer: Andrew Purtell <ap...@apache.org>
Committed: Thu Apr 30 14:03:59 2015 -0700

----------------------------------------------------------------------
 .../regionserver/RegionCoprocessorHost.java     |   8 +-
 .../util/BoundedConcurrentLinkedQueue.java      | 114 +++++++++++++++++++
 .../util/TestBoundedConcurrentLinkedQueue.java  |  85 ++++++++++++++
 3 files changed, 203 insertions(+), 4 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/hbase/blob/786a413e/hbase-server/src/main/java/org/apache/hadoop/hbase/regionserver/RegionCoprocessorHost.java
----------------------------------------------------------------------
diff --git a/hbase-server/src/main/java/org/apache/hadoop/hbase/regionserver/RegionCoprocessorHost.java b/hbase-server/src/main/java/org/apache/hadoop/hbase/regionserver/RegionCoprocessorHost.java
index badf944..92d10e8 100644
--- a/hbase-server/src/main/java/org/apache/hadoop/hbase/regionserver/RegionCoprocessorHost.java
+++ b/hbase-server/src/main/java/org/apache/hadoop/hbase/regionserver/RegionCoprocessorHost.java
@@ -27,8 +27,6 @@ import java.util.List;
 import java.util.Map;
 import java.util.NavigableSet;
 import java.util.UUID;
-import java.util.concurrent.ArrayBlockingQueue;
-import java.util.concurrent.BlockingQueue;
 import java.util.concurrent.ConcurrentHashMap;
 import java.util.concurrent.ConcurrentMap;
 import java.util.regex.Matcher;
@@ -78,6 +76,7 @@ import org.apache.hadoop.hbase.regionserver.compactions.CompactionRequest;
 import org.apache.hadoop.hbase.regionserver.wal.HLogKey;
 import org.apache.hadoop.hbase.wal.WALKey;
 import org.apache.hadoop.hbase.regionserver.wal.WALEdit;
+import org.apache.hadoop.hbase.util.BoundedConcurrentLinkedQueue;
 import org.apache.hadoop.hbase.util.Bytes;
 import org.apache.hadoop.hbase.util.CoprocessorClassLoader;
 import org.apache.hadoop.hbase.util.Pair;
@@ -102,6 +101,7 @@ public class RegionCoprocessorHost
       new ReferenceMap(AbstractReferenceMap.HARD, AbstractReferenceMap.WEAK);
 
   /**
+   * 
    * Encapsulation of the environment of each coprocessor
    */
   static class RegionEnvironment extends CoprocessorHost.Environment
@@ -111,8 +111,8 @@ public class RegionCoprocessorHost
     private RegionServerServices rsServices;
     ConcurrentMap<String, Object> sharedData;
     private static final int LATENCY_BUFFER_SIZE = 100;
-    private final BlockingQueue<Long> coprocessorTimeNanos = new ArrayBlockingQueue<Long>(
-        LATENCY_BUFFER_SIZE);
+    private final BoundedConcurrentLinkedQueue<Long> coprocessorTimeNanos =
+        new BoundedConcurrentLinkedQueue<Long>(LATENCY_BUFFER_SIZE);
     private final boolean useLegacyPre;
     private final boolean useLegacyPost;
 

http://git-wip-us.apache.org/repos/asf/hbase/blob/786a413e/hbase-server/src/main/java/org/apache/hadoop/hbase/util/BoundedConcurrentLinkedQueue.java
----------------------------------------------------------------------
diff --git a/hbase-server/src/main/java/org/apache/hadoop/hbase/util/BoundedConcurrentLinkedQueue.java b/hbase-server/src/main/java/org/apache/hadoop/hbase/util/BoundedConcurrentLinkedQueue.java
new file mode 100644
index 0000000..9208238
--- /dev/null
+++ b/hbase-server/src/main/java/org/apache/hadoop/hbase/util/BoundedConcurrentLinkedQueue.java
@@ -0,0 +1,114 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+package org.apache.hadoop.hbase.util;
+
+import java.util.Collection;
+import java.util.concurrent.ConcurrentLinkedQueue;
+
+import org.apache.hadoop.hbase.classification.InterfaceAudience;
+import org.apache.hadoop.hbase.classification.InterfaceStability;
+
+/**
+ * A ConcurrentLinkedQueue that enforces a maximum queue size.
+ */
+@InterfaceAudience.Private
+@InterfaceStability.Stable
+public class BoundedConcurrentLinkedQueue<T> extends ConcurrentLinkedQueue<T> {
+  private static final long serialVersionUID = 1L;
+  private volatile long size = 0;
+  private final long maxSize;
+
+  public BoundedConcurrentLinkedQueue() {
+    this(Long.MAX_VALUE);
+  }
+
+  public BoundedConcurrentLinkedQueue(long maxSize) {
+    super();
+    this.maxSize = maxSize;
+  }
+
+  @Override
+  public boolean add(T e) {
+    return offer(e);
+  }
+
+  @Override
+  public boolean addAll(Collection<? extends T> c) {
+    size += c.size();        // Between here and below we might reject offers,
+    if (size > maxSize) {    // if over maxSize, but that's ok
+      size -= c.size();      // We're over, just back out and return.
+      return false;
+    }
+    return super.addAll(c);  // Always true for ConcurrentLinkedQueue
+  }
+
+  @Override
+  public void clear() {
+    super.clear();
+    size = 0;
+  }
+
+  @Override
+  public boolean offer(T e) {
+    if (++size > maxSize) {
+      --size;                // We didn't take it after all
+      return false;
+    }
+    return super.offer(e);   // Always true for ConcurrentLinkedQueue
+  }
+
+  @Override
+  public T poll() {
+    T result = super.poll();
+    if (result != null) {
+      --size;
+    }
+    return result;
+  }
+
+  @Override
+  public boolean remove(Object o) {
+    boolean result = super.remove(o);
+    if (result) {
+      --size;
+    }
+    return result;
+  }
+
+  @Override
+  public int size() {
+    return (int) size;
+  }
+
+  public void drainTo(Collection<T> list) {
+    long removed = 0;
+    T l;
+    while ((l = super.poll()) != null) {
+      list.add(l);
+      removed++;
+    }
+    // Limit the number of operations on a volatile by only reporting size
+    // change after the drain is completed.
+    size -= removed;
+  }
+
+  public long remainingCapacity() {
+    long remaining = maxSize - size;
+    return remaining >= 0 ? remaining : 0;
+  }
+}
\ No newline at end of file

http://git-wip-us.apache.org/repos/asf/hbase/blob/786a413e/hbase-server/src/test/java/org/apache/hadoop/hbase/util/TestBoundedConcurrentLinkedQueue.java
----------------------------------------------------------------------
diff --git a/hbase-server/src/test/java/org/apache/hadoop/hbase/util/TestBoundedConcurrentLinkedQueue.java b/hbase-server/src/test/java/org/apache/hadoop/hbase/util/TestBoundedConcurrentLinkedQueue.java
new file mode 100644
index 0000000..5972a87
--- /dev/null
+++ b/hbase-server/src/test/java/org/apache/hadoop/hbase/util/TestBoundedConcurrentLinkedQueue.java
@@ -0,0 +1,85 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one or more
+ * contributor license agreements. See the NOTICE file distributed with this
+ * work for additional information regarding copyright ownership. The ASF
+ * licenses this file to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance with the License.
+ * You may obtain a copy of the License at
+ *
+ * http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS, WITHOUT
+ * WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. See the
+ * License for the specific language governing permissions and limitations
+ * under the License.
+ */
+
+package org.apache.hadoop.hbase.util;
+
+import static org.junit.Assert.assertEquals;
+import static org.junit.Assert.assertTrue;
+import static org.junit.Assert.assertFalse;
+
+import java.util.ArrayList;
+import java.util.List;
+
+import org.apache.hadoop.hbase.testclassification.SmallTests;
+import org.junit.After;
+import org.junit.Before;
+import org.junit.Test;
+import org.junit.experimental.categories.Category;
+
+@Category(SmallTests.class)
+public class TestBoundedConcurrentLinkedQueue {
+  private final static int CAPACITY = 16;
+
+  private BoundedConcurrentLinkedQueue<Long> queue;
+
+  @Before
+  public void setUp() throws Exception {
+    this.queue = new BoundedConcurrentLinkedQueue<Long>(CAPACITY);
+  }
+
+  @After
+  public void tearDown() throws Exception {
+  }
+
+  @Test
+  public void testOfferAndPoll() throws Exception {
+    // Offer
+    for (long i = 1; i <= CAPACITY; ++i) {
+      assertTrue(queue.offer(i));
+      assertEquals(i, queue.size());
+      assertEquals(CAPACITY - i, queue.remainingCapacity());
+    }
+    assertFalse(queue.offer(0L));
+
+    // Poll
+    for (int i = 1; i <= CAPACITY; ++i) {
+      long l = queue.poll();
+      assertEquals(i, l);
+      assertEquals(CAPACITY - i, queue.size());
+      assertEquals(i, queue.remainingCapacity());
+    }
+    assertEquals(null, queue.poll());
+  }
+
+  @Test
+  public void testDrain() throws Exception {
+    // Offer
+    for (long i = 1; i <= CAPACITY; ++i) {
+      assertTrue(queue.offer(i));
+      assertEquals(i, queue.size());
+      assertEquals(CAPACITY - i, queue.remainingCapacity());
+    }
+    assertFalse(queue.offer(0L));
+
+    // Drain
+    List<Long> list = new ArrayList<Long>();
+    queue.drainTo(list);
+    assertEquals(null, queue.poll());
+    assertEquals(0, queue.size());
+    assertEquals(CAPACITY, queue.remainingCapacity());
+  }
+}