This is an automated email from the ASF dual-hosted git repository.

gian pushed a commit to branch master
in repository https://gitbox.apache.org/repos/asf/druid.git


The following commit(s) were added to refs/heads/master by this push:
     new bbe23c6  Fix negative queuedSize problem in CuratorLoadQueuePeon 
(#10362)
bbe23c6 is described below

commit bbe23c652c5810325b1c6b6daa30d8b02244b351
Author: BIGrey <[email protected]>
AuthorDate: Sat Oct 17 04:38:49 2020 +0800

    Fix negative queuedSize problem in CuratorLoadQueuePeon (#10362)
    
    * fix negative queuedSize problem in CuratorLoadQueuePeon
    
    * add comment and optimize test case
    
    * fix typo
    
    Co-authored-by: huagnhui.bigrey <[email protected]>
---
 .../druid/server/coordinator/CuratorLoadQueuePeon.java   |  8 ++++++--
 .../druid/server/coordinator/LoadQueuePeonTest.java      | 16 ++++++++++++++--
 2 files changed, 20 insertions(+), 4 deletions(-)

diff --git 
a/server/src/main/java/org/apache/druid/server/coordinator/CuratorLoadQueuePeon.java
 
b/server/src/main/java/org/apache/druid/server/coordinator/CuratorLoadQueuePeon.java
index 9ca70d4..85573b6 100644
--- 
a/server/src/main/java/org/apache/druid/server/coordinator/CuratorLoadQueuePeon.java
+++ 
b/server/src/main/java/org/apache/druid/server/coordinator/CuratorLoadQueuePeon.java
@@ -302,8 +302,12 @@ public class CuratorLoadQueuePeon extends LoadQueuePeon
   {
     switch (segmentHolder.getType()) {
       case LOAD:
-        segmentsToLoad.remove(segmentHolder.getSegment());
-        queuedSize.addAndGet(-segmentHolder.getSegmentSize());
+        // When load failed a segment will be removed from the segmentsToLoad 
twice and
+        // null value will be returned at the second time in which case 
queueSize may be negative.
+        // See https://github.com/apache/druid/pull/10362 for more details.
+        if (null != segmentsToLoad.remove(segmentHolder.getSegment())) {
+          queuedSize.addAndGet(-segmentHolder.getSegmentSize());
+        }
         break;
       case DROP:
         segmentsToDrop.remove(segmentHolder.getSegment());
diff --git 
a/server/src/test/java/org/apache/druid/server/coordinator/LoadQueuePeonTest.java
 
b/server/src/test/java/org/apache/druid/server/coordinator/LoadQueuePeonTest.java
index 2afad8d..9020264 100644
--- 
a/server/src/test/java/org/apache/druid/server/coordinator/LoadQueuePeonTest.java
+++ 
b/server/src/test/java/org/apache/druid/server/coordinator/LoadQueuePeonTest.java
@@ -274,6 +274,7 @@ public class LoadQueuePeonTest extends CuratorTestBase
 
     final CountDownLatch loadRequestSignal = new CountDownLatch(1);
     final CountDownLatch segmentLoadedSignal = new CountDownLatch(1);
+    final CountDownLatch loadRequestRemoveSignal = new CountDownLatch(1);
 
     loadQueuePeon = new CuratorLoadQueuePeon(
         curator,
@@ -302,8 +303,15 @@ public class LoadQueuePeonTest extends CuratorTestBase
           @Override
           public void childEvent(CuratorFramework client, 
PathChildrenCacheEvent event)
           {
-            if (event.getType() == PathChildrenCacheEvent.Type.CHILD_ADDED) {
-              loadRequestSignal.countDown();
+            switch (event.getType()) {
+              case CHILD_ADDED:
+                loadRequestSignal.countDown();
+                break;
+              case CHILD_REMOVED:
+                loadRequestRemoveSignal.countDown();
+                break;
+              default:
+                // pass
             }
           }
         }
@@ -336,6 +344,10 @@ public class LoadQueuePeonTest extends CuratorTestBase
     Assert.assertTrue(timing.forWaiting().awaitLatch(segmentLoadedSignal));
     Assert.assertEquals(0, loadQueuePeon.getSegmentsToLoad().size());
     Assert.assertEquals(0L, loadQueuePeon.getLoadQueueSize());
+    curator.delete().guaranteed().forPath(loadRequestPath);
+    Assert.assertTrue(timing.forWaiting().awaitLatch(loadRequestRemoveSignal));
+    Assert.assertEquals(0, loadQueuePeon.getSegmentsToLoad().size());
+    Assert.assertEquals(0L, loadQueuePeon.getLoadQueueSize());
   }
 
   private DataSegment dataSegmentWithInterval(String intervalStr)


---------------------------------------------------------------------
To unsubscribe, e-mail: [email protected]
For additional commands, e-mail: [email protected]

Reply via email to