Return-Path: X-Original-To: apmail-ignite-commits-archive@minotaur.apache.org Delivered-To: apmail-ignite-commits-archive@minotaur.apache.org Received: from mail.apache.org (hermes.apache.org [140.211.11.3]) by minotaur.apache.org (Postfix) with SMTP id 1CFBD17E9D for ; Tue, 14 Apr 2015 15:14:05 +0000 (UTC) Received: (qmail 54899 invoked by uid 500); 14 Apr 2015 15:14:02 -0000 Delivered-To: apmail-ignite-commits-archive@ignite.apache.org Received: (qmail 54872 invoked by uid 500); 14 Apr 2015 15:14:02 -0000 Mailing-List: contact commits-help@ignite.incubator.apache.org; run by ezmlm Precedence: bulk List-Help: List-Unsubscribe: List-Post: List-Id: Reply-To: dev@ignite.incubator.apache.org Delivered-To: mailing list commits@ignite.incubator.apache.org Received: (qmail 54863 invoked by uid 99); 14 Apr 2015 15:14:02 -0000 Received: from nike.apache.org (HELO nike.apache.org) (192.87.106.230) by apache.org (qpsmtpd/0.29) with ESMTP; Tue, 14 Apr 2015 15:14:02 +0000 X-ASF-Spam-Status: No, hits=-2000.0 required=5.0 tests=ALL_TRUSTED,T_RP_MATCHES_RCVD X-Spam-Check-By: apache.org Received: from [140.211.11.3] (HELO mail.apache.org) (140.211.11.3) by apache.org (qpsmtpd/0.29) with SMTP; Tue, 14 Apr 2015 15:13:19 +0000 Received: (qmail 52226 invoked by uid 99); 14 Apr 2015 15:13:16 -0000 Received: from git1-us-west.apache.org (HELO git1-us-west.apache.org) (140.211.11.23) by apache.org (qpsmtpd/0.29) with ESMTP; Tue, 14 Apr 2015 15:13:16 +0000 Received: by git1-us-west.apache.org (ASF Mail Server at git1-us-west.apache.org, from userid 33) id 699A9E03CE; Tue, 14 Apr 2015 15:13:16 +0000 (UTC) Content-Type: text/plain; charset="us-ascii" MIME-Version: 1.0 Content-Transfer-Encoding: 7bit From: sboikov@apache.org To: commits@ignite.incubator.apache.org Date: Tue, 14 Apr 2015 15:13:25 -0000 Message-Id: <36b81fc0002a49f7ae012cac1cbbe8ec@git.apache.org> In-Reply-To: <3c9a41f4c747451a8bb289f596b57492@git.apache.org> References: <3c9a41f4c747451a8bb289f596b57492@git.apache.org> X-Mailer: ASF-Git Admin Mailer Subject: [10/50] incubator-ignite git commit: # ignite-sprint-3 reverted ignoring disovery data exchange for daemon nodes (was done in ignite-537) X-Virus-Checked: Checked by ClamAV on apache.org # ignite-sprint-3 reverted ignoring disovery data exchange for daemon nodes (was done in ignite-537) Project: http://git-wip-us.apache.org/repos/asf/incubator-ignite/repo Commit: http://git-wip-us.apache.org/repos/asf/incubator-ignite/commit/46d1369c Tree: http://git-wip-us.apache.org/repos/asf/incubator-ignite/tree/46d1369c Diff: http://git-wip-us.apache.org/repos/asf/incubator-ignite/diff/46d1369c Branch: refs/heads/ignite-714 Commit: 46d1369c2fb17efec325793ced87276b8be98450 Parents: 5953786 Author: sboikov Authored: Sat Apr 11 05:58:13 2015 +0300 Committer: sboikov Committed: Sat Apr 11 06:53:57 2015 +0300 ---------------------------------------------------------------------- .../discovery/tcp/TcpClientDiscoverySpi.java | 4 +- .../spi/discovery/tcp/TcpDiscoverySpi.java | 26 +-- .../cache/CacheConfigurationP2PTest.java | 122 ------------- .../cache/CacheConfigurationP2PTestServer.java | 63 ------- .../CacheConfigurationP2PTestClient.java | 75 ++------ .../tests/p2p/startcache/Organization1.java | 52 ++++++ .../tests/p2p/startcache/Organization2.java | 52 ++++++ .../cache/CacheConfigurationP2PTest.java | 177 +++++++++++++++++++ .../cache/CacheConfigurationP2PTestServer.java | 49 +++++ .../IgniteCacheWithIndexingTestSuite.java | 2 + 10 files changed, 364 insertions(+), 258 deletions(-) ---------------------------------------------------------------------- http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/46d1369c/modules/core/src/main/java/org/apache/ignite/spi/discovery/tcp/TcpClientDiscoverySpi.java ---------------------------------------------------------------------- diff --git a/modules/core/src/main/java/org/apache/ignite/spi/discovery/tcp/TcpClientDiscoverySpi.java b/modules/core/src/main/java/org/apache/ignite/spi/discovery/tcp/TcpClientDiscoverySpi.java index 312b940..2ab8e8f 100644 --- a/modules/core/src/main/java/org/apache/ignite/spi/discovery/tcp/TcpClientDiscoverySpi.java +++ b/modules/core/src/main/java/org/apache/ignite/spi/discovery/tcp/TcpClientDiscoverySpi.java @@ -900,7 +900,7 @@ public class TcpClientDiscoverySpi extends TcpDiscoverySpiAdapter implements Tcp Map> dataMap = msg.oldNodesDiscoveryData(); - if (!locNode.isDaemon() && dataMap != null) { + if (dataMap != null) { for (Map.Entry> entry : dataMap.entrySet()) onExchange(newNodeId, entry.getKey(), entry.getValue(), null); } @@ -924,7 +924,7 @@ public class TcpClientDiscoverySpi extends TcpDiscoverySpiAdapter implements Tcp Map data = msg.newNodeDiscoveryData(); - if (!locNode.isDaemon() && data != null) + if (data != null) onExchange(newNodeId, newNodeId, data, null); } } http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/46d1369c/modules/core/src/main/java/org/apache/ignite/spi/discovery/tcp/TcpDiscoverySpi.java ---------------------------------------------------------------------- diff --git a/modules/core/src/main/java/org/apache/ignite/spi/discovery/tcp/TcpDiscoverySpi.java b/modules/core/src/main/java/org/apache/ignite/spi/discovery/tcp/TcpDiscoverySpi.java index e1b1a92..81c608a 100644 --- a/modules/core/src/main/java/org/apache/ignite/spi/discovery/tcp/TcpDiscoverySpi.java +++ b/modules/core/src/main/java/org/apache/ignite/spi/discovery/tcp/TcpDiscoverySpi.java @@ -2211,7 +2211,7 @@ public class TcpDiscoverySpi extends TcpDiscoverySpiAdapter implements TcpDiscov * @param req Get class request. * @return Get class response. */ - private TcpDiscoveryClassResponse processGetClassRequest(TcpDiscoveryClassRequest req) { + private TcpDiscoveryClassResponse processClassRequest(TcpDiscoveryClassRequest req) { assert !F.isEmpty(req.className()) : req; String rsrc = U.classNameToResourceName(req.className()); @@ -3722,7 +3722,7 @@ public class TcpDiscoverySpi extends TcpDiscoverySpiAdapter implements TcpDiscov Map data = msg.newNodeDiscoveryData(); - if (!locNode.isDaemon() && data != null) + if (data != null) onExchange(node.id(), node.id(), data, exchangeClassLoader(node, node.id())); msg.addDiscoveryData(locNodeId, collectExchangeData(node.id())); @@ -3793,7 +3793,7 @@ public class TcpDiscoverySpi extends TcpDiscoverySpiAdapter implements TcpDiscov } // Notify outside of synchronized block. - if (!locNode.isDaemon() && dataMap != null) { + if (dataMap != null) { for (Map.Entry> entry : dataMap.entrySet()) { onExchange(node.id(), entry.getKey(), @@ -4594,7 +4594,7 @@ public class TcpDiscoverySpi extends TcpDiscoverySpiAdapter implements TcpDiscov Collection snapshot = hist.get(msg.topologyVersion()); - if (!locNode.isDaemon() && lsnr != null && (spiState == CONNECTED || spiState == DISCONNECTING)) { + if (lsnr != null && (spiState == CONNECTED || spiState == DISCONNECTING)) { assert msg.messageBytes() != null; TcpDiscoveryNode node = ring.node(msg.creatorNodeId()); @@ -4604,17 +4604,17 @@ public class TcpDiscoverySpi extends TcpDiscoverySpiAdapter implements TcpDiscov try { if (msgObj == null) msgObj = marsh.unmarshal(msg.messageBytes(), customMessageClassLoader(node)); + + lsnr.onDiscovery(DiscoveryCustomEvent.EVT_DISCOVERY_CUSTOM_EVT, + msg.topologyVersion(), + node, + snapshot, + hist, + msgObj); } catch (IgniteCheckedException e) { - throw new IgniteSpiException("Failed to unmarshal discovery custom message.", e); + U.error(log, "Failed to unmarshal discovery custom message.", e); } - - lsnr.onDiscovery(DiscoveryCustomEvent.EVT_DISCOVERY_CUSTOM_EVT, - msg.topologyVersion(), - node, - snapshot, - hist, - msgObj); } } @@ -5091,7 +5091,7 @@ public class TcpDiscoverySpi extends TcpDiscoverySpiAdapter implements TcpDiscov continue; } else if (msg instanceof TcpDiscoveryClassRequest) { - TcpDiscoveryClassResponse res = processGetClassRequest((TcpDiscoveryClassRequest)msg); + TcpDiscoveryClassResponse res = processClassRequest((TcpDiscoveryClassRequest) msg); writeToSocket(sock, res); http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/46d1369c/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/CacheConfigurationP2PTest.java ---------------------------------------------------------------------- diff --git a/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/CacheConfigurationP2PTest.java b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/CacheConfigurationP2PTest.java deleted file mode 100644 index 96b124c..0000000 --- a/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/CacheConfigurationP2PTest.java +++ /dev/null @@ -1,122 +0,0 @@ -/* - * 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.ignite.internal.processors.cache; - -import org.apache.ignite.internal.util.*; -import org.apache.ignite.internal.util.typedef.*; -import org.apache.ignite.internal.util.typedef.internal.*; -import org.apache.ignite.testframework.junits.common.*; - -import java.util.concurrent.*; - -import static java.util.concurrent.TimeUnit.*; - -/** - * - */ -public class CacheConfigurationP2PTest extends GridCommonAbstractTest { - /** */ - public static final String NODE_START_MSG = "Test external node started"; - - /** */ - private static final String CLIENT_CLS_NAME = - "org.apache.ignite.tests.p2p.startcache.CacheConfigurationP2PTestClient"; - - /** - * @throws Exception If failed. - */ - public void testCacheConfigurationP2P() throws Exception { - final CountDownLatch srvsReadyLatch = new CountDownLatch(2); - - final CountDownLatch clientReadyLatch = new CountDownLatch(1); - - GridJavaProcess node1 = null; - GridJavaProcess node2 = null; - GridJavaProcess clientNode = null; - - try { - node1 = GridJavaProcess.exec( - CacheConfigurationP2PTestServer.class.getName(), null, - log, - new CI1() { - @Override public void apply(String s) { - info("Server node1: " + s); - - if (s.contains(NODE_START_MSG)) - srvsReadyLatch.countDown(); - } - }, - null, - null, - null - ); - - node2 = GridJavaProcess.exec( - CacheConfigurationP2PTestServer.class.getName(), null, - log, - new CI1() { - @Override public void apply(String s) { - info("Server node2: " + s); - - if (s.contains(NODE_START_MSG)) - srvsReadyLatch.countDown(); - } - }, - null, - null, - null - ); - - assertTrue(srvsReadyLatch.await(60, SECONDS)); - - String str = U.getIgniteHome() + "/modules/extdata/p2p/target/classes/"; - - clientNode = GridJavaProcess.exec( - CLIENT_CLS_NAME, null, - log, - new CI1() { - @Override public void apply(String s) { - info("Client node: " + s); - - if (s.contains(NODE_START_MSG)) - clientReadyLatch.countDown(); - } - }, - null, - null, - str - ); - - assertTrue(clientReadyLatch.await(60, SECONDS)); - - int exitCode = clientNode.getProcess().waitFor(); - - assertEquals("Unexpected exit code", 0, exitCode); - } - finally { - if (node1 != null) - node1.killProcess(); - - if (node2 != null) - node2.killProcess(); - - if (clientNode != null) - clientNode.killProcess(); - } - } -} http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/46d1369c/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/CacheConfigurationP2PTestServer.java ---------------------------------------------------------------------- diff --git a/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/CacheConfigurationP2PTestServer.java b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/CacheConfigurationP2PTestServer.java deleted file mode 100644 index 490f951..0000000 --- a/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/CacheConfigurationP2PTestServer.java +++ /dev/null @@ -1,63 +0,0 @@ -/* - * 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.ignite.internal.processors.cache; - -import org.apache.ignite.*; -import org.apache.ignite.configuration.*; -import org.apache.ignite.internal.util.typedef.internal.*; -import org.apache.ignite.spi.discovery.tcp.*; -import org.apache.ignite.spi.discovery.tcp.ipfinder.vm.*; - -import java.util.*; - -/** - * - */ -public class CacheConfigurationP2PTestServer { - /** - * @param args Arguments. - * @throws Exception If failed. - */ - public static void main(String[] args) throws Exception { - System.out.println("Starting test server node."); - - IgniteConfiguration cfg = new IgniteConfiguration(); - - cfg.setPeerClassLoadingEnabled(true); - - cfg.setLocalHost("127.0.0.1"); - - TcpDiscoverySpi disco = new TcpDiscoverySpi(); - - TcpDiscoveryVmIpFinder ipFinder = new TcpDiscoveryVmIpFinder(); - - ipFinder.setAddresses(Arrays.asList("127.0.0.1:47500..47509")); - - disco.setIpFinder(ipFinder); - - cfg.setDiscoverySpi(disco); - - U.setWorkDirectory(null, U.getIgniteHome()); - - try (Ignite ignite = Ignition.start(cfg)) { - System.out.println(CacheConfigurationP2PTest.NODE_START_MSG); - - U.sleep(Long.MAX_VALUE); - } - } -} http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/46d1369c/modules/extdata/p2p/src/main/java/org/apache/ignite/tests/p2p/startcache/CacheConfigurationP2PTestClient.java ---------------------------------------------------------------------- diff --git a/modules/extdata/p2p/src/main/java/org/apache/ignite/tests/p2p/startcache/CacheConfigurationP2PTestClient.java b/modules/extdata/p2p/src/main/java/org/apache/ignite/tests/p2p/startcache/CacheConfigurationP2PTestClient.java index 0634e5f..be3f3bc 100644 --- a/modules/extdata/p2p/src/main/java/org/apache/ignite/tests/p2p/startcache/CacheConfigurationP2PTestClient.java +++ b/modules/extdata/p2p/src/main/java/org/apache/ignite/tests/p2p/startcache/CacheConfigurationP2PTestClient.java @@ -18,13 +18,12 @@ package org.apache.ignite.tests.p2p.startcache; import org.apache.ignite.*; -import org.apache.ignite.cache.query.annotations.*; +import org.apache.ignite.cache.query.*; import org.apache.ignite.configuration.*; import org.apache.ignite.internal.util.typedef.internal.*; import org.apache.ignite.spi.discovery.tcp.*; import org.apache.ignite.spi.discovery.tcp.ipfinder.vm.*; -import java.io.*; import java.util.*; /** @@ -61,7 +60,7 @@ public class CacheConfigurationP2PTestClient { int nodes = ignite.cluster().nodes().size(); - if (nodes != 3) + if (nodes != 4) throw new Exception("Unexpected nodes number: " + nodes); CacheConfiguration ccfg1 = new CacheConfiguration<>(); @@ -79,6 +78,11 @@ public class CacheConfigurationP2PTestClient { for (int i = 0; i < 500; i++) cache1.put(i, new Organization1("org-" + i)); + SqlQuery qry1 = new SqlQuery<>(Organization1.class, "_key >= 0"); + + if (cache1.query(qry1).getAll().isEmpty()) + throw new Exception("Query failed."); + System.out.println("Sleep some time."); Thread.sleep(5000); // Sleep some time to wait when connection of p2p loader is closed. @@ -89,67 +93,22 @@ public class CacheConfigurationP2PTestClient { ccfg2.setName("cache2"); - ccfg2.setIndexedTypes(Integer.class, Organization1.class); + ccfg2.setIndexedTypes(Integer.class, Organization2.class); IgniteCache cache2 = ignite.createCache(ccfg2); - } - } - /** - * Organization class. - */ - private static class Organization1 implements Serializable { - /** Organization ID (indexed). */ - @QuerySqlField(index = true) - private UUID id; - - /** Organization name (indexed). */ - @QuerySqlField(index = true) - private String name; - - /** - * Create organization. - * - * @param name Organization name. - */ - Organization1(String name) { - id = UUID.randomUUID(); - - this.name = name; - } + for (int i = 0; i < 500; i++) + cache2.put(i, new Organization2("org-" + i)); - /** {@inheritDoc} */ - @Override public String toString() { - return "Organization [id=" + id + ", name=" + name + ']'; - } - } + SqlQuery qry2 = new SqlQuery<>(Organization2.class, "_key >= 0"); - /** - * Organization class. - */ - private static class Organization2 implements Serializable { - /** Organization ID (indexed). */ - @QuerySqlField(index = true) - private UUID id; - - /** Organization name (indexed). */ - @QuerySqlField(index = true) - private String name; - - /** - * Create organization. - * - * @param name Organization name. - */ - Organization2(String name) { - id = UUID.randomUUID(); - - this.name = name; - } + if (cache2.query(qry2).getAll().isEmpty()) + throw new Exception("Query failed."); + + cache1.close(); - /** {@inheritDoc} */ - @Override public String toString() { - return "Organization [id=" + id + ", name=" + name + ']'; + cache2.close(); } } + } http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/46d1369c/modules/extdata/p2p/src/main/java/org/apache/ignite/tests/p2p/startcache/Organization1.java ---------------------------------------------------------------------- diff --git a/modules/extdata/p2p/src/main/java/org/apache/ignite/tests/p2p/startcache/Organization1.java b/modules/extdata/p2p/src/main/java/org/apache/ignite/tests/p2p/startcache/Organization1.java new file mode 100644 index 0000000..9c371f1 --- /dev/null +++ b/modules/extdata/p2p/src/main/java/org/apache/ignite/tests/p2p/startcache/Organization1.java @@ -0,0 +1,52 @@ +/* + * 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.ignite.tests.p2p.startcache; + +import org.apache.ignite.cache.query.annotations.*; + +import java.io.*; +import java.util.*; + +/** + * Organization class. + */ +class Organization1 implements Serializable { + /** Organization ID (indexed). */ + @QuerySqlField(index = true) + private UUID id; + + /** Organization name (indexed). */ + @QuerySqlField(index = true) + private String name; + + /** + * Create organization. + * + * @param name Organization name. + */ + Organization1(String name) { + id = UUID.randomUUID(); + + this.name = name; + } + + /** {@inheritDoc} */ + @Override public String toString() { + return "Organization1 [id=" + id + ", name=" + name + ']'; + } +} http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/46d1369c/modules/extdata/p2p/src/main/java/org/apache/ignite/tests/p2p/startcache/Organization2.java ---------------------------------------------------------------------- diff --git a/modules/extdata/p2p/src/main/java/org/apache/ignite/tests/p2p/startcache/Organization2.java b/modules/extdata/p2p/src/main/java/org/apache/ignite/tests/p2p/startcache/Organization2.java new file mode 100644 index 0000000..e44ce97 --- /dev/null +++ b/modules/extdata/p2p/src/main/java/org/apache/ignite/tests/p2p/startcache/Organization2.java @@ -0,0 +1,52 @@ +/* + * 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.ignite.tests.p2p.startcache; + +import org.apache.ignite.cache.query.annotations.*; + +import java.io.*; +import java.util.*; + +/** + * Organization class. + */ +class Organization2 implements Serializable { + /** Organization ID (indexed). */ + @QuerySqlField(index = true) + private UUID id; + + /** Organization name (indexed). */ + @QuerySqlField(index = true) + private String name; + + /** + * Create organization. + * + * @param name Organization name. + */ + Organization2(String name) { + id = UUID.randomUUID(); + + this.name = name; + } + + /** {@inheritDoc} */ + @Override public String toString() { + return "Organization2 [id=" + id + ", name=" + name + ']'; + } +} http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/46d1369c/modules/indexing/src/test/java/org/apache/ignite/internal/processors/cache/CacheConfigurationP2PTest.java ---------------------------------------------------------------------- diff --git a/modules/indexing/src/test/java/org/apache/ignite/internal/processors/cache/CacheConfigurationP2PTest.java b/modules/indexing/src/test/java/org/apache/ignite/internal/processors/cache/CacheConfigurationP2PTest.java new file mode 100644 index 0000000..8f0f4ca --- /dev/null +++ b/modules/indexing/src/test/java/org/apache/ignite/internal/processors/cache/CacheConfigurationP2PTest.java @@ -0,0 +1,177 @@ +/* + * 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.ignite.internal.processors.cache; + +import org.apache.ignite.*; +import org.apache.ignite.configuration.*; +import org.apache.ignite.internal.util.*; +import org.apache.ignite.internal.util.lang.*; +import org.apache.ignite.internal.util.typedef.*; +import org.apache.ignite.internal.util.typedef.internal.*; +import org.apache.ignite.spi.discovery.*; +import org.apache.ignite.spi.discovery.tcp.*; +import org.apache.ignite.spi.discovery.tcp.ipfinder.vm.*; +import org.apache.ignite.testframework.*; +import org.apache.ignite.testframework.junits.common.*; + +import java.util.*; +import java.util.concurrent.*; + +import static java.util.concurrent.TimeUnit.*; + +/** + * + */ +public class CacheConfigurationP2PTest extends GridCommonAbstractTest { + /** */ + public static final String NODE_START_MSG = "Test external node started"; + + /** */ + private static final String CLIENT_CLS_NAME = + "org.apache.ignite.tests.p2p.startcache.CacheConfigurationP2PTestClient"; + + /** + * @return Configuration. + */ + static IgniteConfiguration createConfiguration() { + IgniteConfiguration cfg = new IgniteConfiguration(); + + cfg.setPeerClassLoadingEnabled(true); + + cfg.setLocalHost("127.0.0.1"); + + TcpDiscoverySpi disco = new TcpDiscoverySpi(); + + disco.setIpFinderCleanFrequency(1000); + + TcpDiscoveryVmIpFinder ipFinder = new TcpDiscoveryVmIpFinder(); + + ipFinder.setAddresses(Arrays.asList("127.0.0.1:47500..47509")); + + disco.setIpFinder(ipFinder); + + cfg.setDiscoverySpi(disco); + + return cfg; + } + + /** + * @throws Exception If failed. + */ + public void testCacheConfigurationP2P() throws Exception { + try (Ignite ignite = Ignition.start(createConfiguration())) { + final CountDownLatch srvsReadyLatch = new CountDownLatch(2); + + final CountDownLatch clientReadyLatch = new CountDownLatch(1); + + GridJavaProcess node1 = null; + GridJavaProcess node2 = null; + GridJavaProcess clientNode = null; + + Collection jvmArgs = Arrays.asList("-ea", "-DIGNITE_QUIET=false"); + + try { + node1 = GridJavaProcess.exec( + CacheConfigurationP2PTestServer.class.getName(), + null, + log, + new CI1() { + @Override public void apply(String s) { + info("Server node1: " + s); + + if (s.contains(NODE_START_MSG)) + srvsReadyLatch.countDown(); + } + }, + null, + jvmArgs, + null + ); + + node2 = GridJavaProcess.exec( + CacheConfigurationP2PTestServer.class.getName(), null, + log, + new CI1() { + @Override public void apply(String s) { + info("Server node2: " + s); + + if (s.contains(NODE_START_MSG)) + srvsReadyLatch.countDown(); + } + }, + null, + jvmArgs, + null + ); + + assertTrue(srvsReadyLatch.await(60, SECONDS)); + + String cp = U.getIgniteHome() + "/modules/extdata/p2p/target/classes/"; + + clientNode = GridJavaProcess.exec( + CLIENT_CLS_NAME, null, + log, + new CI1() { + @Override public void apply(String s) { + info("Client node: " + s); + + if (s.contains(NODE_START_MSG)) + clientReadyLatch.countDown(); + } + }, + null, + jvmArgs, + cp + ); + + assertTrue(clientReadyLatch.await(60, SECONDS)); + + int exitCode = clientNode.getProcess().waitFor(); + + assertEquals("Unexpected exit code", 0, exitCode); + + node1.killProcess(); + node2.killProcess(); + + final DiscoverySpi spi = ignite.configuration().getDiscoverySpi(); + + boolean wait = GridTestUtils.waitForCondition(new GridAbsPredicate() { + @Override public boolean apply() { + Map p2pLdrs = GridTestUtils.getFieldValue(spi, "p2pLdrs"); + + log.info("p2pLdrs: " + p2pLdrs.size()); + + return p2pLdrs.isEmpty(); + } + }, 10_000); + + assertTrue(wait); + } + finally { + if (node1 != null) + node1.killProcess(); + + if (node2 != null) + node2.killProcess(); + + if (clientNode != null) + clientNode.killProcess(); + } + } + } +} http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/46d1369c/modules/indexing/src/test/java/org/apache/ignite/internal/processors/cache/CacheConfigurationP2PTestServer.java ---------------------------------------------------------------------- diff --git a/modules/indexing/src/test/java/org/apache/ignite/internal/processors/cache/CacheConfigurationP2PTestServer.java b/modules/indexing/src/test/java/org/apache/ignite/internal/processors/cache/CacheConfigurationP2PTestServer.java new file mode 100644 index 0000000..738f7cb --- /dev/null +++ b/modules/indexing/src/test/java/org/apache/ignite/internal/processors/cache/CacheConfigurationP2PTestServer.java @@ -0,0 +1,49 @@ +/* + * 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.ignite.internal.processors.cache; + +import org.apache.ignite.*; +import org.apache.ignite.configuration.*; +import org.apache.ignite.internal.util.typedef.internal.*; +import org.apache.ignite.spi.discovery.tcp.*; +import org.apache.ignite.spi.discovery.tcp.ipfinder.vm.*; + +import java.util.*; + +/** + * + */ +public class CacheConfigurationP2PTestServer { + /** + * @param args Arguments. + * @throws Exception If failed. + */ + public static void main(String[] args) throws Exception { + System.out.println("Starting test server node."); + + IgniteConfiguration cfg = CacheConfigurationP2PTest.createConfiguration(); + + U.setWorkDirectory(null, U.getIgniteHome()); + + try (Ignite ignite = Ignition.start(cfg)) { + System.out.println(CacheConfigurationP2PTest.NODE_START_MSG); + + U.sleep(Long.MAX_VALUE); + } + } +} http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/46d1369c/modules/indexing/src/test/java/org/apache/ignite/testsuites/IgniteCacheWithIndexingTestSuite.java ---------------------------------------------------------------------- diff --git a/modules/indexing/src/test/java/org/apache/ignite/testsuites/IgniteCacheWithIndexingTestSuite.java b/modules/indexing/src/test/java/org/apache/ignite/testsuites/IgniteCacheWithIndexingTestSuite.java index ba6fe36..a2ccc82 100644 --- a/modules/indexing/src/test/java/org/apache/ignite/testsuites/IgniteCacheWithIndexingTestSuite.java +++ b/modules/indexing/src/test/java/org/apache/ignite/testsuites/IgniteCacheWithIndexingTestSuite.java @@ -46,6 +46,8 @@ public class IgniteCacheWithIndexingTestSuite extends TestSuite { suite.addTestSuite(CacheTtlOnheapAtomicLocalSelfTest.class); suite.addTestSuite(CacheTtlOnheapAtomicPartitionedSelfTest.class); + suite.addTestSuite(CacheConfigurationP2PTest.class); + return suite; } }