You are viewing a plain text version of this content. The canonical link for it is here.
Posted to notifications@geode.apache.org by GitBox <gi...@apache.org> on 2021/03/12 23:22:59 UTC

[GitHub] [geode] boglesby opened a new pull request #6135: GEODE-9030: Modified PartitionedIndex to reset arbitraryBucketIndex

boglesby opened a new pull request #6135:
URL: https://github.com/apache/geode/pull/6135


   Thank you for submitting a contribution to Apache Geode.
   
   In order to streamline the review of the contribution we ask you
   to ensure the following steps have been taken:
   
   ### For all changes:
   - [ ] Is there a JIRA ticket associated with this PR? Is it referenced in the commit message?
   
   - [ ] Has your PR been rebased against the latest commit within the target branch (typically `develop`)?
   
   - [ ] Is your initial contribution a single, squashed commit?
   
   - [ ] Does `gradlew build` run cleanly?
   
   - [ ] Have you written or updated unit tests to verify your changes?
   
   - [ ] If adding new dependencies to the code, are these dependencies licensed in a way that is compatible for inclusion under [ASF 2.0](http://www.apache.org/legal/resolved.html#category-a)?
   
   ### Note:
   Please ensure that once the PR is submitted, check Concourse for build issues and
   submit an update to your PR as soon as possible. If you need help, please send an
   email to dev@geode.apache.org.
   


----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



[GitHub] [geode] DonalEvans commented on a change in pull request #6135: GEODE-9030: Modified PartitionedIndex to reset arbitraryBucketIndex

Posted by GitBox <gi...@apache.org>.
DonalEvans commented on a change in pull request #6135:
URL: https://github.com/apache/geode/pull/6135#discussion_r593512648



##########
File path: geode-core/src/distributedTest/java/org/apache/geode/cache/query/dunit/InequalityQueryWithRebalanceDistributedTest.java
##########
@@ -0,0 +1,172 @@
+/*
+ * 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.geode.cache.query.dunit;
+
+import static org.apache.geode.cache.Region.SEPARATOR;
+
+import java.io.Serializable;
+import java.util.stream.IntStream;
+
+import org.junit.Before;
+import org.junit.BeforeClass;
+import org.junit.ClassRule;
+import org.junit.Rule;
+import org.junit.Test;
+import org.junit.experimental.categories.Category;
+
+import org.apache.geode.cache.EntryOperation;
+import org.apache.geode.cache.PartitionResolver;
+import org.apache.geode.cache.Region;
+import org.apache.geode.pdx.PdxReader;
+import org.apache.geode.pdx.PdxSerializable;
+import org.apache.geode.pdx.PdxWriter;
+import org.apache.geode.test.dunit.rules.ClusterStartupRule;
+import org.apache.geode.test.dunit.rules.MemberVM;
+import org.apache.geode.test.junit.categories.OQLIndexTest;
+import org.apache.geode.test.junit.rules.GfshCommandRule;
+import org.apache.geode.test.junit.rules.serializable.SerializableTestName;
+
+@Category({OQLIndexTest.class})
+public class InequalityQueryWithRebalanceDistributedTest implements Serializable {
+
+  private String regionName;
+
+  private static MemberVM locator;
+
+  @ClassRule
+  public static GfshCommandRule gfsh = new GfshCommandRule();
+
+  @ClassRule
+  public static ClusterStartupRule cluster = new ClusterStartupRule();
+
+  @Rule
+  public SerializableTestName testName = new SerializableTestName();
+
+  @BeforeClass
+  public static void beforeClass() throws Exception {
+    locator = cluster.startLocatorVM(0);
+    gfsh.connectAndVerify(locator);
+  }
+
+  @Before
+  public void before() throws Exception {
+    regionName = getClass().getSimpleName() + "_" + testName.getMethodName() + "_region";
+  }
+
+  @Test
+  public void testArbitraryBucketIndexUpdatedAfterBucketMoved() throws Exception {

Review comment:
       This `throws Exception` can be removed.

##########
File path: geode-core/src/distributedTest/java/org/apache/geode/cache/query/dunit/InequalityQueryWithRebalanceDistributedTest.java
##########
@@ -0,0 +1,172 @@
+/*
+ * 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.geode.cache.query.dunit;
+
+import static org.apache.geode.cache.Region.SEPARATOR;
+
+import java.io.Serializable;
+import java.util.stream.IntStream;
+
+import org.junit.Before;
+import org.junit.BeforeClass;
+import org.junit.ClassRule;
+import org.junit.Rule;
+import org.junit.Test;
+import org.junit.experimental.categories.Category;
+
+import org.apache.geode.cache.EntryOperation;
+import org.apache.geode.cache.PartitionResolver;
+import org.apache.geode.cache.Region;
+import org.apache.geode.pdx.PdxReader;
+import org.apache.geode.pdx.PdxSerializable;
+import org.apache.geode.pdx.PdxWriter;
+import org.apache.geode.test.dunit.rules.ClusterStartupRule;
+import org.apache.geode.test.dunit.rules.MemberVM;
+import org.apache.geode.test.junit.categories.OQLIndexTest;
+import org.apache.geode.test.junit.rules.GfshCommandRule;
+import org.apache.geode.test.junit.rules.serializable.SerializableTestName;
+
+@Category({OQLIndexTest.class})
+public class InequalityQueryWithRebalanceDistributedTest implements Serializable {
+
+  private String regionName;
+
+  private static MemberVM locator;
+
+  @ClassRule
+  public static GfshCommandRule gfsh = new GfshCommandRule();
+
+  @ClassRule
+  public static ClusterStartupRule cluster = new ClusterStartupRule();
+
+  @Rule
+  public SerializableTestName testName = new SerializableTestName();
+
+  @BeforeClass
+  public static void beforeClass() throws Exception {
+    locator = cluster.startLocatorVM(0);
+    gfsh.connectAndVerify(locator);
+  }
+
+  @Before
+  public void before() throws Exception {
+    regionName = getClass().getSimpleName() + "_" + testName.getMethodName() + "_region";
+  }
+
+  @Test
+  public void testArbitraryBucketIndexUpdatedAfterBucketMoved() throws Exception {
+    // Start server1
+    MemberVM server1 = cluster.startServerVM(1, locator.getPort());
+
+    // Create the region
+    createRegion();
+
+    // Create the index
+    createIndex();
+
+    // Load entries
+    server1.invoke(() -> loadRegion());
+
+    // Start server2
+    MemberVM server2 = cluster.startServerVM(2, locator.getPort());

Review comment:
       This can be just `cluster.startServerVM(2, locator.getPort());` since the variable `server2` is never used.

##########
File path: geode-core/src/distributedTest/java/org/apache/geode/cache/query/dunit/InequalityQueryWithRebalanceDistributedTest.java
##########
@@ -0,0 +1,172 @@
+/*
+ * 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.geode.cache.query.dunit;
+
+import static org.apache.geode.cache.Region.SEPARATOR;
+
+import java.io.Serializable;
+import java.util.stream.IntStream;
+
+import org.junit.Before;
+import org.junit.BeforeClass;
+import org.junit.ClassRule;
+import org.junit.Rule;
+import org.junit.Test;
+import org.junit.experimental.categories.Category;
+
+import org.apache.geode.cache.EntryOperation;
+import org.apache.geode.cache.PartitionResolver;
+import org.apache.geode.cache.Region;
+import org.apache.geode.pdx.PdxReader;
+import org.apache.geode.pdx.PdxSerializable;
+import org.apache.geode.pdx.PdxWriter;
+import org.apache.geode.test.dunit.rules.ClusterStartupRule;
+import org.apache.geode.test.dunit.rules.MemberVM;
+import org.apache.geode.test.junit.categories.OQLIndexTest;
+import org.apache.geode.test.junit.rules.GfshCommandRule;
+import org.apache.geode.test.junit.rules.serializable.SerializableTestName;
+
+@Category({OQLIndexTest.class})
+public class InequalityQueryWithRebalanceDistributedTest implements Serializable {
+
+  private String regionName;
+
+  private static MemberVM locator;
+
+  @ClassRule
+  public static GfshCommandRule gfsh = new GfshCommandRule();
+
+  @ClassRule
+  public static ClusterStartupRule cluster = new ClusterStartupRule();
+
+  @Rule
+  public SerializableTestName testName = new SerializableTestName();
+
+  @BeforeClass
+  public static void beforeClass() throws Exception {
+    locator = cluster.startLocatorVM(0);
+    gfsh.connectAndVerify(locator);
+  }
+
+  @Before
+  public void before() throws Exception {
+    regionName = getClass().getSimpleName() + "_" + testName.getMethodName() + "_region";
+  }
+
+  @Test
+  public void testArbitraryBucketIndexUpdatedAfterBucketMoved() throws Exception {
+    // Start server1
+    MemberVM server1 = cluster.startServerVM(1, locator.getPort());
+
+    // Create the region
+    createRegion();
+
+    // Create the index
+    createIndex();
+
+    // Load entries
+    server1.invoke(() -> loadRegion());
+
+    // Start server2
+    MemberVM server2 = cluster.startServerVM(2, locator.getPort());
+
+    // Wait for server2 to create the region MBean
+    locator.waitUntilRegionIsReadyOnExactlyThisManyServers(SEPARATOR + regionName, 2);
+
+    // Invoke rebalance
+    rebalance();
+
+    // Execute query
+    executeQuery();
+  }
+
+  private void createRegion() {
+    gfsh.executeAndAssertThat("create region --name=" + regionName
+        + " --type=PARTITION  --total-num-buckets=2 --partition-resolver="
+        + IsolatingPartitionResolver.class.getName()).statusIsSuccess();
+    locator.waitUntilRegionIsReadyOnExactlyThisManyServers(SEPARATOR + regionName, 1);
+  }
+
+  private void createIndex() {
+    gfsh.executeAndAssertThat("create index --region=" + regionName
+        + " --name=tradeStatus --expression=tradeStatus.toString()").statusIsSuccess();
+  }
+
+  private void loadRegion() {
+    Region region = ClusterStartupRule.getCache().getRegion(regionName);

Review comment:
       Compiler warnings here can be fixed by using `Region<Integer, Trade> region`

##########
File path: geode-core/src/distributedTest/java/org/apache/geode/cache/query/dunit/InequalityQueryWithRebalanceDistributedTest.java
##########
@@ -0,0 +1,172 @@
+/*
+ * 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.geode.cache.query.dunit;
+
+import static org.apache.geode.cache.Region.SEPARATOR;
+
+import java.io.Serializable;
+import java.util.stream.IntStream;
+
+import org.junit.Before;
+import org.junit.BeforeClass;
+import org.junit.ClassRule;
+import org.junit.Rule;
+import org.junit.Test;
+import org.junit.experimental.categories.Category;
+
+import org.apache.geode.cache.EntryOperation;
+import org.apache.geode.cache.PartitionResolver;
+import org.apache.geode.cache.Region;
+import org.apache.geode.pdx.PdxReader;
+import org.apache.geode.pdx.PdxSerializable;
+import org.apache.geode.pdx.PdxWriter;
+import org.apache.geode.test.dunit.rules.ClusterStartupRule;
+import org.apache.geode.test.dunit.rules.MemberVM;
+import org.apache.geode.test.junit.categories.OQLIndexTest;
+import org.apache.geode.test.junit.rules.GfshCommandRule;
+import org.apache.geode.test.junit.rules.serializable.SerializableTestName;
+
+@Category({OQLIndexTest.class})
+public class InequalityQueryWithRebalanceDistributedTest implements Serializable {
+
+  private String regionName;
+
+  private static MemberVM locator;
+
+  @ClassRule
+  public static GfshCommandRule gfsh = new GfshCommandRule();
+
+  @ClassRule
+  public static ClusterStartupRule cluster = new ClusterStartupRule();
+
+  @Rule
+  public SerializableTestName testName = new SerializableTestName();
+
+  @BeforeClass
+  public static void beforeClass() throws Exception {
+    locator = cluster.startLocatorVM(0);
+    gfsh.connectAndVerify(locator);
+  }
+
+  @Before
+  public void before() throws Exception {
+    regionName = getClass().getSimpleName() + "_" + testName.getMethodName() + "_region";
+  }
+
+  @Test
+  public void testArbitraryBucketIndexUpdatedAfterBucketMoved() throws Exception {
+    // Start server1
+    MemberVM server1 = cluster.startServerVM(1, locator.getPort());
+
+    // Create the region
+    createRegion();
+
+    // Create the index
+    createIndex();
+
+    // Load entries
+    server1.invoke(() -> loadRegion());
+
+    // Start server2
+    MemberVM server2 = cluster.startServerVM(2, locator.getPort());
+
+    // Wait for server2 to create the region MBean
+    locator.waitUntilRegionIsReadyOnExactlyThisManyServers(SEPARATOR + regionName, 2);
+
+    // Invoke rebalance
+    rebalance();
+
+    // Execute query
+    executeQuery();
+  }
+
+  private void createRegion() {
+    gfsh.executeAndAssertThat("create region --name=" + regionName
+        + " --type=PARTITION  --total-num-buckets=2 --partition-resolver="
+        + IsolatingPartitionResolver.class.getName()).statusIsSuccess();
+    locator.waitUntilRegionIsReadyOnExactlyThisManyServers(SEPARATOR + regionName, 1);
+  }
+
+  private void createIndex() {
+    gfsh.executeAndAssertThat("create index --region=" + regionName
+        + " --name=tradeStatus --expression=tradeStatus.toString()").statusIsSuccess();
+  }
+
+  private void loadRegion() {
+    Region region = ClusterStartupRule.getCache().getRegion(regionName);
+    IntStream.range(0, 10).forEach(i -> region.put(i,
+        new Trade(String.valueOf(i), "aId1", i % 2 == 0 ? TradeStatus.OPEN : TradeStatus.CLOSED)));
+  }
+
+  private void rebalance() {
+    gfsh.executeAndAssertThat("rebalance").statusIsSuccess();
+  }
+
+  private void executeQuery() {
+    gfsh.executeAndAssertThat("query --query=\"SELECT * FROM " + SEPARATOR + regionName
+        + " WHERE arrangementId = 'aId1' AND tradeStatus.toString() != 'CLOSED'\"")
+        .statusIsSuccess();
+  }
+
+  public enum TradeStatus {
+    OPEN,
+    CLOSED;

Review comment:
       Unnecessary semicolon here.

##########
File path: geode-core/src/integrationTest/java/org/apache/geode/cache/query/internal/index/PartitionedIndexJUnitTest.java
##########
@@ -66,4 +62,37 @@ public void mapIndexKeysMustContainTheCorrectNumberOfKeysWhenThereIsConcurrentAc
     assertEquals(DATA_SIZE_TO_BE_POPULATED, partitionedIndex.mapIndexKeys.size());
 
   }
+
+  @Test
+  public void verifyAddToAndRemoveFromBucketIndexesUpdatesArbitraryBucketIndex() {

Review comment:
       Would it be possible to also add a test case (or expand this one slightly) that shows that if there are multiple indexes and the current `arbitraryBucketIndex` is removed, that the value of `arbitraryBucketIndex` is reset to another existing index rather than null?

##########
File path: geode-core/src/integrationTest/java/org/apache/geode/cache/query/internal/index/PartitionedIndexJUnitTest.java
##########
@@ -66,4 +62,37 @@ public void mapIndexKeysMustContainTheCorrectNumberOfKeysWhenThereIsConcurrentAc
     assertEquals(DATA_SIZE_TO_BE_POPULATED, partitionedIndex.mapIndexKeys.size());
 
   }
+
+  @Test
+  public void verifyAddToAndRemoveFromBucketIndexesUpdatesArbitraryBucketIndex() {
+    // Create the PartitionedIndex
+    PartitionedIndex partitionedIndex = createPartitionedIndex();
+
+    // Create the mock Region and Index
+    Region region = mock(Region.class);
+    Index index = mock(Index.class);
+
+    // Add the mock region and index to the bucket indexes
+    partitionedIndex.addToBucketIndexes(region, index);
+
+    // Assert that the arbitraryBucketIndex is set
+    assertThat(partitionedIndex.getBucketIndex()).isEqualTo(index);
+
+    // Remove the mock region and index from the bucket indexes
+    partitionedIndex.removeFromBucketIndexes(region, index);
+
+    // Assert that the arbitraryBucketIndex is null
+    assertThat(partitionedIndex.getBucketIndex()).isNull();
+  }
+
+  private PartitionedIndex createPartitionedIndex() {
+    Region region = mock(Region.class);
+    InternalCache cache = mock(InternalCache.class);
+    when(region.getCache()).thenReturn(cache);
+    DistributedSystem distributedSystem = mock(DistributedSystem.class);
+    when(cache.getDistributedSystem()).thenReturn(distributedSystem);
+    PartitionedIndex partitionedIndex = new PartitionedIndex(cache, IndexType.FUNCTIONAL,
+        "dummyString", region, "dummyString", "dummyString", "dummyString");
+    return partitionedIndex;

Review comment:
       This can be simplified to just `return new PartitionedIndex(cache, IndexType.FUNCTIONAL,
           "dummyString", region, "dummyString", "dummyString", "dummyString");`




----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



[GitHub] [geode] boglesby merged pull request #6135: GEODE-9030: Modified PartitionedIndex to reset arbitraryBucketIndex

Posted by GitBox <gi...@apache.org>.
boglesby merged pull request #6135:
URL: https://github.com/apache/geode/pull/6135


   


----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org