IGNITE-9018 PDS1 TC configuration hangs periodically

Signed-off-by: EdShangGG <[email protected]> - Fixes #4373.
Signed-off-by: Dmitriy Pavlov <[email protected]>


Project: http://git-wip-us.apache.org/repos/asf/ignite/repo
Commit: http://git-wip-us.apache.org/repos/asf/ignite/commit/174e9cb5
Tree: http://git-wip-us.apache.org/repos/asf/ignite/tree/174e9cb5
Diff: http://git-wip-us.apache.org/repos/asf/ignite/diff/174e9cb5

Branch: refs/heads/ignite-8783
Commit: 174e9cb5506268e764b9b542c6cc22d3f92dd06e
Parents: 46db052
Author: EdShangGG <[email protected]>
Authored: Tue Jul 17 16:58:17 2018 +0300
Committer: Dmitriy Pavlov <[email protected]>
Committed: Tue Jul 17 16:59:31 2018 +0300

----------------------------------------------------------------------
 .../BPlusTreeReuseListPageMemoryImplTest.java   |   4 +-
 .../wal/SegmentedRingByteBufferTest.java        | 346 ++++++++++---------
 .../processors/database/BPlusTreeSelfTest.java  |   3 +-
 3 files changed, 184 insertions(+), 169 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/ignite/blob/174e9cb5/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/persistence/pagemem/BPlusTreeReuseListPageMemoryImplTest.java
----------------------------------------------------------------------
diff --git 
a/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/persistence/pagemem/BPlusTreeReuseListPageMemoryImplTest.java
 
b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/persistence/pagemem/BPlusTreeReuseListPageMemoryImplTest.java
index 0c786ad..c36ecce 100644
--- 
a/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/persistence/pagemem/BPlusTreeReuseListPageMemoryImplTest.java
+++ 
b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/persistence/pagemem/BPlusTreeReuseListPageMemoryImplTest.java
@@ -23,11 +23,13 @@ import 
org.apache.ignite.internal.mem.unsafe.UnsafeMemoryProvider;
 import org.apache.ignite.internal.pagemem.FullPageId;
 import org.apache.ignite.internal.pagemem.PageMemory;
 import org.apache.ignite.internal.processors.cache.GridCacheSharedContext;
+import 
org.apache.ignite.internal.processors.cache.persistence.CheckpointWriteProgressSupplier;
 import 
org.apache.ignite.internal.processors.cache.persistence.DataRegionMetricsImpl;
 import 
org.apache.ignite.internal.processors.cache.persistence.IgniteCacheDatabaseSharedManager;
 import org.apache.ignite.internal.processors.database.BPlusTreeReuseSelfTest;
 import org.apache.ignite.internal.util.lang.GridInClosure3X;
 import org.apache.ignite.testframework.junits.GridTestKernalContext;
+import org.mockito.Mockito;
 
 /**
  *
@@ -77,7 +79,7 @@ public class BPlusTreeReuseListPageMemoryImplTest extends 
BPlusTreeReuseSelfTest
             () -> true,
             new DataRegionMetricsImpl(new DataRegionConfiguration()),
             PageMemoryImpl.ThrottlingPolicy.DISABLED,
-            null
+            Mockito.mock(CheckpointWriteProgressSupplier.class)
         );
 
         mem.start();

http://git-wip-us.apache.org/repos/asf/ignite/blob/174e9cb5/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/persistence/wal/SegmentedRingByteBufferTest.java
----------------------------------------------------------------------
diff --git 
a/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/persistence/wal/SegmentedRingByteBufferTest.java
 
b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/persistence/wal/SegmentedRingByteBufferTest.java
index 3a24738..72563a2 100644
--- 
a/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/persistence/wal/SegmentedRingByteBufferTest.java
+++ 
b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/persistence/wal/SegmentedRingByteBufferTest.java
@@ -313,87 +313,91 @@ public class SegmentedRingByteBufferTest extends TestCase 
{
 
         final Object mux = new Object();
 
-        IgniteInternalFuture<Long> fut = 
GridTestUtils.runMultiThreadedAsync(() -> {
-            try {
-                barrier.await();
-            }
-            catch (InterruptedException | BrokenBarrierException e) {
-                e.printStackTrace();
+        IgniteInternalFuture<Long> fut;
 
-                fail();
-            }
+        try {
+            fut = GridTestUtils.runMultiThreadedAsync(() -> {
+                try {
+                    barrier.await();
+                }
+                catch (InterruptedException | BrokenBarrierException e) {
+                    e.printStackTrace();
 
-            while (!stop.get()) {
-                TestObject obj = new TestObject();
+                    fail();
+                }
 
-                SegmentedRingByteBuffer.WriteSegment seg = 
buf.offer(obj.size());
-                ByteBuffer bbuf;
+                while (!stop.get()) {
+                    TestObject obj = new TestObject();
 
-                if (seg == null) {
-                    cnt.incrementAndGet();
+                    SegmentedRingByteBuffer.WriteSegment seg = 
buf.offer(obj.size());
+                    ByteBuffer bbuf;
 
-                    synchronized (mux) {
-                        try {
-                            if (stop.get())
-                                break;
+                    if (seg == null) {
+                        cnt.incrementAndGet();
 
-                            mux.wait();
-                        }
-                        catch (InterruptedException e) {
-                            e.printStackTrace();
+                        synchronized (mux) {
+                            try {
+                                if (stop.get())
+                                    break;
+
+                                mux.wait();
+                            }
+                            catch (InterruptedException e) {
+                                e.printStackTrace();
+                            }
                         }
+
+                        cnt.decrementAndGet();
+
+                        continue;
                     }
 
-                    cnt.decrementAndGet();
+                    bbuf = seg.buffer();
 
-                    continue;
-                }
+                    assertEquals(obj.size(), bbuf.remaining());
 
-                bbuf = seg.buffer();
+                    bbuf.putLong(obj.id);
+                    bbuf.putInt(obj.len);
+                    bbuf.put(obj.arr);
 
-                assertEquals(obj.size(), bbuf.remaining());
+                    assertEquals(0, bbuf.remaining());
 
-                bbuf.putLong(obj.id);
-                bbuf.putInt(obj.len);
-                bbuf.put(obj.arr);
+                    seg.release();
 
-                assertEquals(0, bbuf.remaining());
+                    long total = totalWritten.addAndGet(obj.size());
 
-                seg.release();
+                    assertTrue(total <= cap);
+                }
+            }, producerCnt, "producer-thread");
 
-                long total = totalWritten.addAndGet(obj.size());
+            long endTime = System.currentTimeMillis() + 60 * 1000L;
 
-                assertTrue(total <= cap);
-            }
-        }, producerCnt, "producer-thread");
+            while (System.currentTimeMillis() < endTime) {
 
-        long endTime = System.currentTimeMillis() + 60 * 1000L;
+                while (cnt.get() < producerCnt)
+                    U.sleep(10);
 
-        while (System.currentTimeMillis() < endTime) {
+                synchronized (mux) {
+                    List<SegmentedRingByteBuffer.ReadSegment> segs = 
buf.poll();
 
-            while (cnt.get() < producerCnt)
-                U.sleep(10);
+                    if (segs != null) {
+                        for (SegmentedRingByteBuffer.ReadSegment seg : segs)
+                            seg.release();
+                    }
 
-            synchronized (mux) {
-                List<SegmentedRingByteBuffer.ReadSegment> segs = buf.poll();
+                    totalWritten.set(0);
 
-                if (segs != null) {
-                    for (SegmentedRingByteBuffer.ReadSegment seg : segs)
-                        seg.release();
+                    mux.notifyAll();
                 }
-
-                totalWritten.set(0);
+            }
+        } finally {
+            synchronized (mux) {
+                stop.set(true);
 
                 mux.notifyAll();
             }
         }
 
-        synchronized (mux) {
-            stop.set(true);
-
-            mux.notifyAll();
-        }
-
         fut.get();
     }
 
@@ -411,71 +415,75 @@ public class SegmentedRingByteBufferTest extends TestCase 
{
 
         final CyclicBarrier barrier = new CyclicBarrier(producerCnt);
 
-        IgniteInternalFuture<Long> fut = 
GridTestUtils.runMultiThreadedAsync(() -> {
-            try {
-                barrier.await();
-            }
-            catch (InterruptedException | BrokenBarrierException e) {
-                e.printStackTrace();
+        IgniteInternalFuture<Long> fut;
 
-                fail();
-            }
+        try {
+            fut = GridTestUtils.runMultiThreadedAsync(() -> {
+                try {
+                    barrier.await();
+                }
+                catch (InterruptedException | BrokenBarrierException e) {
+                    e.printStackTrace();
 
-            while (!stop.get()) {
-                TestObject obj = new TestObject();
+                    fail();
+                }
 
-                SegmentedRingByteBuffer.WriteSegment seg;
-                ByteBuffer bbuf;
+                while (!stop.get()) {
+                    TestObject obj = new TestObject();
 
-                for (;;) {
-                    if (stop.get())
-                        return;
+                    SegmentedRingByteBuffer.WriteSegment seg;
+                    ByteBuffer bbuf;
 
-                    seg = buf.offer(obj.size());
+                    for (;;) {
+                        if (stop.get())
+                            return;
 
-                    if (seg != null)
-                        break;
-                }
+                        seg = buf.offer(obj.size());
 
-                bbuf = seg.buffer();
+                        if (seg != null)
+                            break;
+                    }
 
-                assertEquals(obj.size(), bbuf.remaining());
+                    bbuf = seg.buffer();
 
-                bbuf.putLong(obj.id);
-                bbuf.putInt(obj.len);
-                bbuf.put(obj.arr);
+                    assertEquals(obj.size(), bbuf.remaining());
 
-                assertEquals(0, bbuf.remaining());
+                    bbuf.putLong(obj.id);
+                    bbuf.putInt(obj.len);
+                    bbuf.put(obj.arr);
 
-                seg.release();
-            }
-        }, producerCnt, "producer-thread");
+                    assertEquals(0, bbuf.remaining());
 
-        Random rnd = new Random();
+                    seg.release();
+                }
+            }, producerCnt, "producer-thread");
 
-        long endTime = System.currentTimeMillis() + 60 * 1000L;
+            Random rnd = new Random();
 
-        while (System.currentTimeMillis() < endTime) {
-            try {
-                U.sleep(rnd.nextInt(100) + 1);
-            }
-            catch (IgniteInterruptedCheckedException e) {
-                e.printStackTrace();
-            }
+            long endTime = System.currentTimeMillis() + 60 * 1000L;
 
-            List<SegmentedRingByteBuffer.ReadSegment> segs;
+            while (System.currentTimeMillis() < endTime) {
+                try {
+                    U.sleep(rnd.nextInt(100) + 1);
+                }
+                catch (IgniteInterruptedCheckedException e) {
+                    e.printStackTrace();
+                }
 
-            if ((segs = buf.poll()) != null) {
-                for (SegmentedRingByteBuffer.ReadSegment seg : segs) {
-                    assertTrue(seg.buffer().hasRemaining());
+                List<SegmentedRingByteBuffer.ReadSegment> segs;
 
-                    seg.release();
+                if ((segs = buf.poll()) != null) {
+                    for (SegmentedRingByteBuffer.ReadSegment seg : segs) {
+                        assertTrue(seg.buffer().hasRemaining());
+
+                        seg.release();
+                    }
                 }
             }
+        } finally {
+            stop.set(true);
         }
 
-        stop.set(true);
-
         fut.get();
     }
 
@@ -495,116 +503,120 @@ public class SegmentedRingByteBufferTest extends 
TestCase {
 
         final Set<TestObject> items = Collections.newSetFromMap(new 
ConcurrentHashMap<TestObject, Boolean>());
 
-        IgniteInternalFuture<Long> fut = 
GridTestUtils.runMultiThreadedAsync(() -> {
-            try {
-                barrier.await();
-            }
-            catch (InterruptedException | BrokenBarrierException e) {
-                e.printStackTrace();
+        IgniteInternalFuture<Long> fut;
 
-                fail();
-            }
+        try {
+            fut = GridTestUtils.runMultiThreadedAsync(() -> {
+                try {
+                    barrier.await();
+                }
+                catch (InterruptedException | BrokenBarrierException e) {
+                    e.printStackTrace();
 
-            while (!stop.get()) {
-                TestObject obj = new TestObject();
+                    fail();
+                }
 
-                SegmentedRingByteBuffer.WriteSegment seg;
-                ByteBuffer bbuf;
+                while (!stop.get()) {
+                    TestObject obj = new TestObject();
 
-                for (;;) {
-                    if (stop.get())
-                        return;
+                    SegmentedRingByteBuffer.WriteSegment seg;
+                    ByteBuffer bbuf;
 
-                    seg = buf.offer(obj.size());
+                    for (;;) {
+                        if (stop.get())
+                            return;
 
-                    if (seg != null)
-                        break;
-                }
+                        seg = buf.offer(obj.size());
 
-                bbuf = seg.buffer();
+                        if (seg != null)
+                            break;
+                    }
 
-                assertEquals(obj.size(), bbuf.remaining());
+                    bbuf = seg.buffer();
 
-                bbuf.putLong(obj.id);
-                bbuf.putInt(obj.len);
-                bbuf.put(obj.arr);
+                    assertEquals(obj.size(), bbuf.remaining());
 
-                assertEquals(0, bbuf.remaining());
+                    bbuf.putLong(obj.id);
+                    bbuf.putInt(obj.len);
+                    bbuf.put(obj.arr);
 
-                assertTrue("Ooops! The same value is already exist in Set! ", 
items.add(obj));
+                    assertEquals(0, bbuf.remaining());
 
-                seg.release();
-            }
-        }, producerCnt, "producer-thread");
+                    assertTrue("Ooops! The same value is already exist in Set! 
", items.add(obj));
 
-        Random rnd = new Random();
+                    seg.release();
+                }
+            }, producerCnt, "producer-thread");
 
-        long endTime = System.currentTimeMillis() + 60 * 1000L;
+            Random rnd = new Random();
 
-        while (System.currentTimeMillis() < endTime) {
-            try {
-                U.sleep(rnd.nextInt(100) + 1);
-            }
-            catch (IgniteInterruptedCheckedException e) {
-                e.printStackTrace();
-            }
+            long endTime = System.currentTimeMillis() + 60 * 1000L;
 
-            List<SegmentedRingByteBuffer.ReadSegment> segs;
+            while (System.currentTimeMillis() < endTime) {
+                try {
+                    U.sleep(rnd.nextInt(100) + 1);
+                }
+                catch (IgniteInterruptedCheckedException e) {
+                    e.printStackTrace();
+                }
 
-            while ((segs = buf.poll()) != null) {
-                int size = 0;
+                List<SegmentedRingByteBuffer.ReadSegment> segs;
 
-                for (SegmentedRingByteBuffer.ReadSegment seg : segs) {
-                    ByteBuffer bbuf = seg.buffer();
+                while ((segs = buf.poll()) != null) {
+                    int size = 0;
 
-                    assertTrue(bbuf.hasRemaining());
+                    for (SegmentedRingByteBuffer.ReadSegment seg : segs) {
+                        ByteBuffer bbuf = seg.buffer();
 
-                    size += bbuf.remaining();
-                }
+                        assertTrue(bbuf.hasRemaining());
 
-                byte[] arr = new byte[size];
+                        size += bbuf.remaining();
+                    }
 
-                int idx = 0;
+                    byte[] arr = new byte[size];
 
-                for (SegmentedRingByteBuffer.ReadSegment seg : segs) {
-                    ByteBuffer bbuf = seg.buffer();
+                    int idx = 0;
 
-                    assertTrue(bbuf.hasRemaining());
+                    for (SegmentedRingByteBuffer.ReadSegment seg : segs) {
+                        ByteBuffer bbuf = seg.buffer();
 
-                    int len = bbuf.remaining();
+                        assertTrue(bbuf.hasRemaining());
 
-                    bbuf.get(arr, idx, len);
+                        int len = bbuf.remaining();
 
-                    idx += len;
-                }
+                        bbuf.get(arr, idx, len);
+
+                        idx += len;
+                    }
 
-                ByteBuffer bbuf = ByteBuffer.wrap(arr);
+                    ByteBuffer bbuf = ByteBuffer.wrap(arr);
 
-                bbuf.order(ByteOrder.nativeOrder());
+                    bbuf.order(ByteOrder.nativeOrder());
 
-                assertTrue(bbuf.hasRemaining());
+                    assertTrue(bbuf.hasRemaining());
 
-                while (bbuf.hasRemaining()) {
-                    long id = bbuf.getLong();
+                    while (bbuf.hasRemaining()) {
+                        long id = bbuf.getLong();
 
-                    int len = bbuf.getInt();
+                        int len = bbuf.getInt();
 
-                    arr = new byte[len];
+                        arr = new byte[len];
 
-                    bbuf.get(arr);
+                        bbuf.get(arr);
 
-                    TestObject obj = new TestObject(id, arr);
+                        TestObject obj = new TestObject(id, arr);
 
-                    assertTrue(items.remove(obj));
-                }
+                        assertTrue(items.remove(obj));
+                    }
 
-                for (SegmentedRingByteBuffer.ReadSegment seg : segs)
-                    seg.release();
+                    for (SegmentedRingByteBuffer.ReadSegment seg : segs)
+                        seg.release();
+                }
             }
+        } finally {
+            stop.set(true);
         }
 
-        stop.set(true);
-
         fut.get();
 
         List<SegmentedRingByteBuffer.ReadSegment> segs;

http://git-wip-us.apache.org/repos/asf/ignite/blob/174e9cb5/modules/core/src/test/java/org/apache/ignite/internal/processors/database/BPlusTreeSelfTest.java
----------------------------------------------------------------------
diff --git 
a/modules/core/src/test/java/org/apache/ignite/internal/processors/database/BPlusTreeSelfTest.java
 
b/modules/core/src/test/java/org/apache/ignite/internal/processors/database/BPlusTreeSelfTest.java
index eeb3373..bd9a099 100644
--- 
a/modules/core/src/test/java/org/apache/ignite/internal/processors/database/BPlusTreeSelfTest.java
+++ 
b/modules/core/src/test/java/org/apache/ignite/internal/processors/database/BPlusTreeSelfTest.java
@@ -201,7 +201,8 @@ public class BPlusTreeSelfTest extends 
GridCommonAbstractTest {
             assertEquals(0, acquiredPages());
         }
         finally {
-            pageMem.stop();
+            if (pageMem != null)
+                pageMem.stop();
 
             MAX_PER_PAGE = 0;
             PUT_INC = 1;

Reply via email to