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 3A88E17796 for ; Wed, 10 Jun 2015 14:12:37 +0000 (UTC) Received: (qmail 28581 invoked by uid 500); 10 Jun 2015 14:12:37 -0000 Delivered-To: apmail-ignite-commits-archive@ignite.apache.org Received: (qmail 28549 invoked by uid 500); 10 Jun 2015 14:12:37 -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 28540 invoked by uid 99); 10 Jun 2015 14:12:37 -0000 Received: from nike.apache.org (HELO nike.apache.org) (192.87.106.230) by apache.org (qpsmtpd/0.29) with ESMTP; Wed, 10 Jun 2015 14:12:37 +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; Wed, 10 Jun 2015 14:09:37 +0000 Received: (qmail 23250 invoked by uid 99); 10 Jun 2015 14:11:23 -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; Wed, 10 Jun 2015 14:11:23 +0000 Received: by git1-us-west.apache.org (ASF Mail Server at git1-us-west.apache.org, from userid 33) id 1630EE045C; Wed, 10 Jun 2015 14:11:23 +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: Wed, 10 Jun 2015 14:11:33 -0000 Message-Id: <55996e83699c45b1943dfa7de6a10334@git.apache.org> In-Reply-To: References: X-Mailer: ASF-Git Admin Mailer Subject: [12/28] incubator-ignite git commit: ignite-545: merge from sprint-6 X-Virus-Checked: Checked by ClamAV on apache.org http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/1652fd18/modules/core/src/test/java/org/apache/ignite/cache/store/CacheStoreSessionListenerAbstractSelfTest.java ---------------------------------------------------------------------- diff --git a/modules/core/src/test/java/org/apache/ignite/cache/store/CacheStoreSessionListenerAbstractSelfTest.java b/modules/core/src/test/java/org/apache/ignite/cache/store/CacheStoreSessionListenerAbstractSelfTest.java new file mode 100644 index 0000000..adac0b2 --- /dev/null +++ b/modules/core/src/test/java/org/apache/ignite/cache/store/CacheStoreSessionListenerAbstractSelfTest.java @@ -0,0 +1,315 @@ +/* + * 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.cache.store; + +import org.apache.ignite.*; +import org.apache.ignite.cache.*; +import org.apache.ignite.configuration.*; +import org.apache.ignite.internal.util.typedef.*; +import org.apache.ignite.spi.discovery.tcp.*; +import org.apache.ignite.spi.discovery.tcp.ipfinder.*; +import org.apache.ignite.spi.discovery.tcp.ipfinder.vm.*; +import org.apache.ignite.testframework.junits.common.*; +import org.apache.ignite.transactions.*; + +import javax.cache.configuration.*; +import javax.cache.integration.*; +import java.io.*; +import java.sql.*; +import java.util.concurrent.atomic.*; + +/** + * Tests for store session listeners. + */ +public abstract class CacheStoreSessionListenerAbstractSelfTest extends GridCommonAbstractTest implements Serializable { + /** */ + private static final TcpDiscoveryIpFinder IP_FINDER = new TcpDiscoveryVmIpFinder(true); + + /** */ + protected static final String URL = "jdbc:h2:mem:example;DB_CLOSE_DELAY=-1"; + + /** */ + protected static final AtomicInteger loadCacheCnt = new AtomicInteger(); + + /** */ + protected static final AtomicInteger loadCnt = new AtomicInteger(); + + /** */ + protected static final AtomicInteger writeCnt = new AtomicInteger(); + + /** */ + protected static final AtomicInteger deleteCnt = new AtomicInteger(); + + /** */ + protected static final AtomicInteger reuseCnt = new AtomicInteger(); + + /** */ + protected static final AtomicBoolean write = new AtomicBoolean(); + + /** */ + protected static final AtomicBoolean fail = new AtomicBoolean(); + + /** {@inheritDoc} */ + @Override protected IgniteConfiguration getConfiguration(String gridName) throws Exception { + IgniteConfiguration cfg = super.getConfiguration(gridName); + + TcpDiscoverySpi disco = new TcpDiscoverySpi(); + + disco.setIpFinder(IP_FINDER); + + cfg.setDiscoverySpi(disco); + + return cfg; + } + + /** {@inheritDoc} */ + @Override protected void beforeTestsStarted() throws Exception { + startGridsMultiThreaded(3); + } + + /** {@inheritDoc} */ + @Override protected void afterTestsStopped() throws Exception { + stopAllGrids(); + } + + /** {@inheritDoc} */ + @Override protected void beforeTest() throws Exception { + try (Connection conn = DriverManager.getConnection(URL)) { + conn.createStatement().executeUpdate("DROP TABLE IF EXISTS Table1"); + conn.createStatement().executeUpdate("DROP TABLE IF EXISTS Table2"); + + conn.createStatement().executeUpdate("CREATE TABLE Table1 (id INT AUTO_INCREMENT, key INT, value INT)"); + conn.createStatement().executeUpdate("CREATE TABLE Table2 (id INT AUTO_INCREMENT, key INT, value INT)"); + } + + loadCacheCnt.set(0); + loadCnt.set(0); + writeCnt.set(0); + deleteCnt.set(0); + reuseCnt.set(0); + + write.set(false); + fail.set(false); + } + + /** + * @throws Exception If failed. + */ + public void testAtomicCache() throws Exception { + CacheConfiguration cfg = cacheConfiguration(null, CacheAtomicityMode.ATOMIC); + + try (IgniteCache cache = ignite(0).createCache(cfg)) { + cache.loadCache(null); + cache.get(1); + cache.put(1, 1); + cache.remove(1); + } + + assertEquals(3, loadCacheCnt.get()); + assertEquals(1, loadCnt.get()); + assertEquals(1, writeCnt.get()); + assertEquals(1, deleteCnt.get()); + assertEquals(0, reuseCnt.get()); + } + + /** + * @throws Exception If failed. + */ + public void testTransactionalCache() throws Exception { + CacheConfiguration cfg = cacheConfiguration(null, CacheAtomicityMode.TRANSACTIONAL); + + try (IgniteCache cache = ignite(0).createCache(cfg)) { + cache.loadCache(null); + cache.get(1); + cache.put(1, 1); + cache.remove(1); + } + + assertEquals(3, loadCacheCnt.get()); + assertEquals(1, loadCnt.get()); + assertEquals(1, writeCnt.get()); + assertEquals(1, deleteCnt.get()); + assertEquals(0, reuseCnt.get()); + + } + + /** + * @throws Exception If failed. + */ + public void testExplicitTransaction() throws Exception { + CacheConfiguration cfg = cacheConfiguration(null, CacheAtomicityMode.TRANSACTIONAL); + + try (IgniteCache cache = ignite(0).createCache(cfg)) { + try (Transaction tx = ignite(0).transactions().txStart()) { + cache.put(1, 1); + cache.put(2, 2); + cache.remove(3); + cache.remove(4); + + tx.commit(); + } + } + + assertEquals(2, writeCnt.get()); + assertEquals(2, deleteCnt.get()); + assertEquals(3, reuseCnt.get()); + } + + /** + * @throws Exception If failed. + */ + public void testCrossCacheTransaction() throws Exception { + CacheConfiguration cfg1 = cacheConfiguration("cache1", CacheAtomicityMode.TRANSACTIONAL); + CacheConfiguration cfg2 = cacheConfiguration("cache2", CacheAtomicityMode.TRANSACTIONAL); + + try ( + IgniteCache cache1 = ignite(0).createCache(cfg1); + IgniteCache cache2 = ignite(0).createCache(cfg2) + ) { + try (Transaction tx = ignite(0).transactions().txStart()) { + cache1.put(1, 1); + cache2.put(2, 2); + cache1.remove(3); + cache2.remove(4); + + tx.commit(); + } + } + + assertEquals(2, writeCnt.get()); + assertEquals(2, deleteCnt.get()); + assertEquals(3, reuseCnt.get()); + } + + /** + * @throws Exception If failed. + */ + public void testCommit() throws Exception { + write.set(true); + + CacheConfiguration cfg1 = cacheConfiguration("cache1", CacheAtomicityMode.TRANSACTIONAL); + CacheConfiguration cfg2 = cacheConfiguration("cache2", CacheAtomicityMode.TRANSACTIONAL); + + try ( + IgniteCache cache1 = ignite(0).createCache(cfg1); + IgniteCache cache2 = ignite(0).createCache(cfg2) + ) { + try (Transaction tx = ignite(0).transactions().txStart()) { + cache1.put(1, 1); + cache2.put(2, 2); + + tx.commit(); + } + } + + try (Connection conn = DriverManager.getConnection(URL)) { + checkTable(conn, 1, false); + checkTable(conn, 2, false); + } + } + + /** + * @throws Exception If failed. + */ + public void testRollback() throws Exception { + write.set(true); + fail.set(true); + + CacheConfiguration cfg1 = cacheConfiguration("cache1", CacheAtomicityMode.TRANSACTIONAL); + CacheConfiguration cfg2 = cacheConfiguration("cache2", CacheAtomicityMode.TRANSACTIONAL); + + try ( + IgniteCache cache1 = ignite(0).createCache(cfg1); + IgniteCache cache2 = ignite(0).createCache(cfg2) + ) { + try (Transaction tx = ignite(0).transactions().txStart()) { + cache1.put(1, 1); + cache2.put(2, 2); + + tx.commit(); + + assert false : "Exception was not thrown."; + } + catch (IgniteException e) { + CacheWriterException we = X.cause(e, CacheWriterException.class); + + assertNotNull(we); + + assertEquals("Expected failure.", we.getMessage()); + } + } + + try (Connection conn = DriverManager.getConnection(URL)) { + checkTable(conn, 1, true); + checkTable(conn, 2, true); + } + } + + /** + * @param conn Connection. + * @param idx Table index. + * @param empty If table expected to be empty. + * @throws Exception In case of error. + */ + private void checkTable(Connection conn, int idx, boolean empty) throws Exception { + ResultSet rs = conn.createStatement().executeQuery("SELECT key, value FROM Table" + idx); + + int cnt = 0; + + while (rs.next()) { + int key = rs.getInt(1); + int val = rs.getInt(2); + + assertEquals(idx, key); + assertEquals(idx, val); + + cnt++; + } + + assertEquals(empty ? 0 : 1, cnt); + } + + /** + * @param name Cache name. + * @param atomicity Atomicity mode. + * @return Cache configuration. + */ + private CacheConfiguration cacheConfiguration(String name, CacheAtomicityMode atomicity) { + CacheConfiguration cfg = new CacheConfiguration<>(); + + cfg.setName(name); + cfg.setAtomicityMode(atomicity); + cfg.setCacheStoreFactory(storeFactory()); + cfg.setCacheStoreSessionListenerFactories(sessionListenerFactory()); + cfg.setReadThrough(true); + cfg.setWriteThrough(true); + cfg.setLoadPreviousValue(true); + + return cfg; + } + + /** + * @return Store factory. + */ + protected abstract Factory> storeFactory(); + + /** + * @return Session listener factory. + */ + protected abstract Factory sessionListenerFactory(); +} http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/1652fd18/modules/core/src/test/java/org/apache/ignite/cache/store/CacheStoreSessionListenerLifecycleSelfTest.java ---------------------------------------------------------------------- diff --git a/modules/core/src/test/java/org/apache/ignite/cache/store/CacheStoreSessionListenerLifecycleSelfTest.java b/modules/core/src/test/java/org/apache/ignite/cache/store/CacheStoreSessionListenerLifecycleSelfTest.java new file mode 100644 index 0000000..814c8a5 --- /dev/null +++ b/modules/core/src/test/java/org/apache/ignite/cache/store/CacheStoreSessionListenerLifecycleSelfTest.java @@ -0,0 +1,395 @@ +/* + * 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.cache.store; + +import org.apache.ignite.*; +import org.apache.ignite.configuration.*; +import org.apache.ignite.lifecycle.*; +import org.apache.ignite.resources.*; +import org.apache.ignite.spi.discovery.tcp.*; +import org.apache.ignite.spi.discovery.tcp.ipfinder.*; +import org.apache.ignite.spi.discovery.tcp.ipfinder.vm.*; +import org.apache.ignite.testframework.junits.common.*; +import org.apache.ignite.transactions.*; + +import javax.cache.*; +import javax.cache.configuration.*; +import javax.cache.integration.*; +import java.util.*; +import java.util.concurrent.*; + +import static org.apache.ignite.cache.CacheAtomicityMode.*; + +/** + * Store session listeners test. + */ +public class CacheStoreSessionListenerLifecycleSelfTest extends GridCommonAbstractTest { + /** */ + private static final TcpDiscoveryIpFinder IP_FINDER = new TcpDiscoveryVmIpFinder(true); + + /** */ + private static final Queue evts = new ConcurrentLinkedDeque<>(); + + /** {@inheritDoc} */ + @Override protected IgniteConfiguration getConfiguration(String gridName) throws Exception { + IgniteConfiguration cfg = super.getConfiguration(gridName); + + cfg.setCacheStoreSessionListenerFactories( + new SessionListenerFactory("Shared 1"), + new SessionListenerFactory("Shared 2") + ); + + TcpDiscoverySpi disco = new TcpDiscoverySpi(); + + disco.setIpFinder(IP_FINDER); + + cfg.setDiscoverySpi(disco); + + return cfg; + } + + /** {@inheritDoc} */ + @Override protected void beforeTest() throws Exception { + evts.clear(); + } + + /** + * @throws Exception If failed. + */ + public void testNoCaches() throws Exception { + try { + startGrid(); + } + finally { + stopGrid(); + } + + assertEqualsCollections(Arrays.asList("Shared 1 START", "Shared 2 START", "Shared 1 STOP", "Shared 2 STOP"), + evts); + } + + /** + * @throws Exception If failed. + */ + public void testNoOverride() throws Exception { + try { + Ignite ignite = startGrid(); + + for (int i = 0; i < 2; i++) { + CacheConfiguration cacheCfg = cacheConfiguration("cache-" + i); + + cacheCfg.setAtomicityMode(TRANSACTIONAL); + + ignite.createCache(cacheCfg); + } + + ignite.cache("cache-0").put(1, 1); + ignite.cache("cache-1").put(1, 1); + + try (Transaction tx = ignite.transactions().txStart()) { + ignite.cache("cache-0").put(2, 2); + ignite.cache("cache-0").put(3, 3); + ignite.cache("cache-1").put(2, 2); + ignite.cache("cache-1").put(3, 3); + + tx.commit(); + } + } + finally { + stopGrid(); + } + + assertEqualsCollections(Arrays.asList( + "Shared 1 START", + "Shared 2 START", + + // Put to cache-0. + "Shared 1 SESSION START cache-0", + "Shared 2 SESSION START cache-0", + "Shared 1 SESSION END cache-0", + "Shared 2 SESSION END cache-0", + + // Put to cache-1. + "Shared 1 SESSION START cache-1", + "Shared 2 SESSION START cache-1", + "Shared 1 SESSION END cache-1", + "Shared 2 SESSION END cache-1", + + // Transaction. + "Shared 1 SESSION START cache-0", + "Shared 2 SESSION START cache-0", + "Shared 1 SESSION START cache-1", + "Shared 2 SESSION START cache-1", + "Shared 1 SESSION END cache-0", + "Shared 2 SESSION END cache-0", + "Shared 1 SESSION END cache-1", + "Shared 2 SESSION END cache-1", + + "Shared 1 STOP", + "Shared 2 STOP" + ), evts); + } + + /** + * @throws Exception If failed. + */ + public void testPartialOverride() throws Exception { + try { + Ignite ignite = startGrid(); + + for (int i = 0; i < 2; i++) { + String name = "cache-" + i; + + CacheConfiguration cacheCfg = cacheConfiguration(name); + + cacheCfg.setAtomicityMode(TRANSACTIONAL); + + if (i == 0) { + cacheCfg.setCacheStoreSessionListenerFactories( + new SessionListenerFactory(name + " 1"), + new SessionListenerFactory(name + " 2") + ); + } + + ignite.createCache(cacheCfg); + } + + ignite.cache("cache-0").put(1, 1); + ignite.cache("cache-1").put(1, 1); + + try (Transaction tx = ignite.transactions().txStart()) { + ignite.cache("cache-0").put(2, 2); + ignite.cache("cache-0").put(3, 3); + ignite.cache("cache-1").put(2, 2); + ignite.cache("cache-1").put(3, 3); + + tx.commit(); + } + } + finally { + stopGrid(); + } + + assertEqualsCollections(Arrays.asList( + "Shared 1 START", + "Shared 2 START", + "cache-0 1 START", + "cache-0 2 START", + + // Put to cache-0. + "cache-0 1 SESSION START cache-0", + "cache-0 2 SESSION START cache-0", + "cache-0 1 SESSION END cache-0", + "cache-0 2 SESSION END cache-0", + + // Put to cache-1. + "Shared 1 SESSION START cache-1", + "Shared 2 SESSION START cache-1", + "Shared 1 SESSION END cache-1", + "Shared 2 SESSION END cache-1", + + // Transaction. + "cache-0 1 SESSION START cache-0", + "cache-0 2 SESSION START cache-0", + "Shared 1 SESSION START cache-1", + "Shared 2 SESSION START cache-1", + "cache-0 1 SESSION END cache-0", + "cache-0 2 SESSION END cache-0", + "Shared 1 SESSION END cache-1", + "Shared 2 SESSION END cache-1", + + "cache-0 1 STOP", + "cache-0 2 STOP", + "Shared 1 STOP", + "Shared 2 STOP" + ), evts); + } + + /** + * @throws Exception If failed. + */ + public void testOverride() throws Exception { + try { + Ignite ignite = startGrid(); + + for (int i = 0; i < 2; i++) { + String name = "cache-" + i; + + CacheConfiguration cacheCfg = cacheConfiguration(name); + + cacheCfg.setCacheStoreSessionListenerFactories(new SessionListenerFactory(name + " 1"), new SessionListenerFactory(name + " 2")); + + ignite.createCache(cacheCfg); + } + + ignite.cache("cache-0").put(1, 1); + ignite.cache("cache-1").put(1, 1); + + try (Transaction tx = ignite.transactions().txStart()) { + ignite.cache("cache-0").put(2, 2); + ignite.cache("cache-0").put(3, 3); + ignite.cache("cache-1").put(2, 2); + ignite.cache("cache-1").put(3, 3); + + tx.commit(); + } + } + finally { + stopGrid(); + } + + assertEqualsCollections(Arrays.asList( + "Shared 1 START", + "Shared 2 START", + "cache-0 1 START", + "cache-0 2 START", + "cache-1 1 START", + "cache-1 2 START", + + // Put to cache-0. + "cache-0 1 SESSION START cache-0", + "cache-0 2 SESSION START cache-0", + "cache-0 1 SESSION END cache-0", + "cache-0 2 SESSION END cache-0", + + // Put to cache-1. + "cache-1 1 SESSION START cache-1", + "cache-1 2 SESSION START cache-1", + "cache-1 1 SESSION END cache-1", + "cache-1 2 SESSION END cache-1", + + // Transaction. + "cache-0 1 SESSION START cache-0", + "cache-0 2 SESSION START cache-0", + "cache-1 1 SESSION START cache-1", + "cache-1 2 SESSION START cache-1", + "cache-0 1 SESSION END cache-0", + "cache-0 2 SESSION END cache-0", + "cache-1 1 SESSION END cache-1", + "cache-1 2 SESSION END cache-1", + + "cache-0 1 STOP", + "cache-0 2 STOP", + "cache-1 1 STOP", + "cache-1 2 STOP", + "Shared 1 STOP", + "Shared 2 STOP" + ), evts); + } + + /** + * @param name Cache name. + * @return Cache configuration. + */ + private CacheConfiguration cacheConfiguration(String name) { + CacheConfiguration cacheCfg = new CacheConfiguration<>(name); + + cacheCfg.setAtomicityMode(TRANSACTIONAL); + cacheCfg.setCacheStoreFactory(FactoryBuilder.factoryOf(Store.class)); + cacheCfg.setWriteThrough(true); + + return cacheCfg; + } + + /** + */ + private static class SessionListener implements CacheStoreSessionListener, LifecycleAware { + /** */ + private final String name; + + /** */ + @IgniteInstanceResource + private Ignite ignite; + + /** + * @param name Name. + */ + private SessionListener(String name) { + this.name = name; + } + + /** {@inheritDoc} */ + @Override public void start() throws IgniteException { + assertNotNull(ignite); + + evts.add(name + " START"); + } + + /** {@inheritDoc} */ + @Override public void stop() throws IgniteException { + assertNotNull(ignite); + + evts.add(name + " STOP"); + } + + /** {@inheritDoc} */ + @Override public void onSessionStart(CacheStoreSession ses) { + assertNotNull(ignite); + + evts.add(name + " SESSION START " + ses.cacheName()); + } + + /** {@inheritDoc} */ + @Override public void onSessionEnd(CacheStoreSession ses, boolean commit) { + assertNotNull(ignite); + + evts.add(name + " SESSION END " + ses.cacheName()); + } + } + + /** + */ + private static class SessionListenerFactory implements Factory { + /** */ + private String name; + + /** + * @param name Name. + */ + private SessionListenerFactory(String name) { + this.name = name; + } + + @Override public CacheStoreSessionListener create() { + return new SessionListener(name); + } + } + + /** + */ + public static class Store extends CacheStoreAdapter { + public Store() { + } + + /** {@inheritDoc} */ + @Override public Integer load(Integer key) throws CacheLoaderException { + return null; + } + + /** {@inheritDoc} */ + @Override public void write(Cache.Entry entry) + throws CacheWriterException { + // No-op. + } + + /** {@inheritDoc} */ + @Override public void delete(Object key) throws CacheWriterException { + // No-op. + } + } +} http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/1652fd18/modules/core/src/test/java/org/apache/ignite/cache/store/jdbc/CacheJdbcStoreSessionListenerSelfTest.java ---------------------------------------------------------------------- diff --git a/modules/core/src/test/java/org/apache/ignite/cache/store/jdbc/CacheJdbcStoreSessionListenerSelfTest.java b/modules/core/src/test/java/org/apache/ignite/cache/store/jdbc/CacheJdbcStoreSessionListenerSelfTest.java new file mode 100644 index 0000000..64af249 --- /dev/null +++ b/modules/core/src/test/java/org/apache/ignite/cache/store/jdbc/CacheJdbcStoreSessionListenerSelfTest.java @@ -0,0 +1,175 @@ +/* + * 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.cache.store.jdbc; + +import org.apache.ignite.cache.store.*; +import org.apache.ignite.lang.*; +import org.apache.ignite.resources.*; +import org.h2.jdbcx.*; + +import javax.cache.*; +import javax.cache.configuration.*; +import javax.cache.integration.*; +import java.sql.*; +import java.util.*; + +/** + * Tests for {@link CacheJdbcStoreSessionListener}. + */ +public class CacheJdbcStoreSessionListenerSelfTest extends CacheStoreSessionListenerAbstractSelfTest { + /** {@inheritDoc} */ + @Override protected Factory> storeFactory() { + return new Factory>() { + @Override public CacheStore create() { + return new Store(); + } + }; + } + + /** {@inheritDoc} */ + @Override protected Factory sessionListenerFactory() { + return new Factory() { + @Override public CacheStoreSessionListener create() { + CacheJdbcStoreSessionListener lsnr = new CacheJdbcStoreSessionListener(); + + lsnr.setDataSource(JdbcConnectionPool.create(URL, "", "")); + + return lsnr; + } + }; + } + + /** + */ + private static class Store extends CacheStoreAdapter { + /** */ + private static String SES_CONN_KEY = "ses_conn"; + + /** */ + @CacheStoreSessionResource + private CacheStoreSession ses; + + /** {@inheritDoc} */ + @Override public void loadCache(IgniteBiInClosure clo, Object... args) { + loadCacheCnt.incrementAndGet(); + + checkConnection(); + } + + /** {@inheritDoc} */ + @Override public Integer load(Integer key) throws CacheLoaderException { + loadCnt.incrementAndGet(); + + checkConnection(); + + return null; + } + + /** {@inheritDoc} */ + @Override public void write(Cache.Entry entry) + throws CacheWriterException { + writeCnt.incrementAndGet(); + + checkConnection(); + + if (write.get()) { + Connection conn = ses.attachment(); + + try { + String table; + + switch (ses.cacheName()) { + case "cache1": + table = "Table1"; + + break; + + case "cache2": + if (fail.get()) + throw new CacheWriterException("Expected failure."); + + table = "Table2"; + + break; + + default: + throw new CacheWriterException("Wring cache: " + ses.cacheName()); + } + + PreparedStatement stmt = conn.prepareStatement( + "INSERT INTO " + table + " (key, value) VALUES (?, ?)"); + + stmt.setInt(1, entry.getKey()); + stmt.setInt(2, entry.getValue()); + + stmt.executeUpdate(); + } + catch (SQLException e) { + throw new CacheWriterException(e); + } + } + } + + /** {@inheritDoc} */ + @Override public void delete(Object key) throws CacheWriterException { + deleteCnt.incrementAndGet(); + + checkConnection(); + } + + /** {@inheritDoc} */ + @Override public void sessionEnd(boolean commit) { + assertNull(ses.attachment()); + } + + /** + */ + private void checkConnection() { + Connection conn = ses.attachment(); + + assertNotNull(conn); + + try { + assertFalse(conn.isClosed()); + assertFalse(conn.getAutoCommit()); + } + catch (SQLException e) { + throw new RuntimeException(e); + } + + verifySameInstance(conn); + } + + /** + * @param conn Connection. + */ + private void verifySameInstance(Connection conn) { + Map props = ses.properties(); + + Connection sesConn = props.get(SES_CONN_KEY); + + if (sesConn == null) + props.put(SES_CONN_KEY, conn); + else { + assertSame(conn, sesConn); + + reuseCnt.incrementAndGet(); + } + } + } +} http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/1652fd18/modules/core/src/test/java/org/apache/ignite/internal/GridAffinitySelfTest.java ---------------------------------------------------------------------- diff --git a/modules/core/src/test/java/org/apache/ignite/internal/GridAffinitySelfTest.java b/modules/core/src/test/java/org/apache/ignite/internal/GridAffinitySelfTest.java index 6735021..7bc8dd6 100644 --- a/modules/core/src/test/java/org/apache/ignite/internal/GridAffinitySelfTest.java +++ b/modules/core/src/test/java/org/apache/ignite/internal/GridAffinitySelfTest.java @@ -47,6 +47,7 @@ public class GridAffinitySelfTest extends GridCommonAbstractTest { disco.setMaxMissedHeartbeats(Integer.MAX_VALUE); disco.setIpFinder(IP_FINDER); + disco.setForceServerMode(true); cfg.setDiscoverySpi(disco); http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/1652fd18/modules/core/src/test/java/org/apache/ignite/internal/GridDiscoveryEventSelfTest.java ---------------------------------------------------------------------- diff --git a/modules/core/src/test/java/org/apache/ignite/internal/GridDiscoveryEventSelfTest.java b/modules/core/src/test/java/org/apache/ignite/internal/GridDiscoveryEventSelfTest.java index 59a4f5d..d90bafe 100644 --- a/modules/core/src/test/java/org/apache/ignite/internal/GridDiscoveryEventSelfTest.java +++ b/modules/core/src/test/java/org/apache/ignite/internal/GridDiscoveryEventSelfTest.java @@ -33,7 +33,7 @@ import java.util.*; import java.util.concurrent.*; import java.util.concurrent.atomic.*; -import static java.util.concurrent.TimeUnit.MILLISECONDS; +import static java.util.concurrent.TimeUnit.*; import static org.apache.ignite.events.EventType.*; /** @@ -163,8 +163,6 @@ public class GridDiscoveryEventSelfTest extends GridCommonAbstractTest { private AtomicInteger cnt = new AtomicInteger(); @Override public boolean apply(Event evt) { - assert evt.type() == EVT_NODE_LEFT; - evts.put(cnt.getAndIncrement(), ((DiscoveryEvent) evt).topologyNodes()); latch.countDown(); @@ -228,7 +226,8 @@ public class GridDiscoveryEventSelfTest extends GridCommonAbstractTest { private AtomicInteger cnt = new AtomicInteger(); @Override public boolean apply(Event evt) { - assert evt.type() == EVT_NODE_JOINED || evt.type() == EVT_NODE_LEFT; + assert evt.type() == EVT_NODE_JOINED + || evt.type() == EVT_NODE_LEFT || evt.type() == EVT_NODE_FAILED; evts.put(cnt.getAndIncrement(), ((DiscoveryEvent) evt).topologyNodes()); http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/1652fd18/modules/core/src/test/java/org/apache/ignite/internal/GridFailFastNodeFailureDetectionSelfTest.java ---------------------------------------------------------------------- diff --git a/modules/core/src/test/java/org/apache/ignite/internal/GridFailFastNodeFailureDetectionSelfTest.java b/modules/core/src/test/java/org/apache/ignite/internal/GridFailFastNodeFailureDetectionSelfTest.java index 46512fe..a116d54 100644 --- a/modules/core/src/test/java/org/apache/ignite/internal/GridFailFastNodeFailureDetectionSelfTest.java +++ b/modules/core/src/test/java/org/apache/ignite/internal/GridFailFastNodeFailureDetectionSelfTest.java @@ -33,6 +33,9 @@ import org.apache.ignite.testframework.junits.common.*; import java.util.*; import java.util.concurrent.*; +import static java.util.concurrent.TimeUnit.*; +import static org.apache.ignite.events.EventType.*; + /** * Fail fast test. */ @@ -76,7 +79,7 @@ public class GridFailFastNodeFailureDetectionSelfTest extends GridCommonAbstract return true; } - }, EventType.EVT_NODE_FAILED); + }, EVT_NODE_FAILED); } Ignite ignite1 = ignite(0); @@ -98,7 +101,7 @@ public class GridFailFastNodeFailureDetectionSelfTest extends GridCommonAbstract failNode(ignite1); - assert failLatch.await(500, TimeUnit.MILLISECONDS); + assert failLatch.await(1000, MILLISECONDS); } /** http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/1652fd18/modules/core/src/test/java/org/apache/ignite/internal/GridProjectionAbstractTest.java ---------------------------------------------------------------------- diff --git a/modules/core/src/test/java/org/apache/ignite/internal/GridProjectionAbstractTest.java b/modules/core/src/test/java/org/apache/ignite/internal/GridProjectionAbstractTest.java index 0171290..cb1341c 100644 --- a/modules/core/src/test/java/org/apache/ignite/internal/GridProjectionAbstractTest.java +++ b/modules/core/src/test/java/org/apache/ignite/internal/GridProjectionAbstractTest.java @@ -20,9 +20,13 @@ package org.apache.ignite.internal; import org.apache.ignite.*; import org.apache.ignite.cluster.*; import org.apache.ignite.compute.*; +import org.apache.ignite.configuration.*; import org.apache.ignite.events.*; import org.apache.ignite.internal.util.typedef.*; import org.apache.ignite.lang.*; +import org.apache.ignite.spi.discovery.tcp.*; +import org.apache.ignite.spi.discovery.tcp.ipfinder.*; +import org.apache.ignite.spi.discovery.tcp.ipfinder.vm.*; import org.apache.ignite.testframework.junits.common.*; import org.jetbrains.annotations.*; @@ -38,6 +42,9 @@ import static org.apache.ignite.events.EventType.*; */ @SuppressWarnings("deprecation") public abstract class GridProjectionAbstractTest extends GridCommonAbstractTest implements Externalizable { + /** VM ip finder for TCP discovery. */ + private static TcpDiscoveryIpFinder ipFinder = new TcpDiscoveryVmIpFinder(true); + /** Waiting timeout. */ private static final int WAIT_TIMEOUT = 30000; @@ -87,6 +94,15 @@ public abstract class GridProjectionAbstractTest extends GridCommonAbstractTest // No-op. } + /** {@inheritDoc} */ + @Override protected IgniteConfiguration getConfiguration(String gridName) throws Exception { + IgniteConfiguration cfg = super.getConfiguration(gridName); + + cfg.setDiscoverySpi(new TcpDiscoverySpi().setForceServerMode(true).setIpFinder(ipFinder)); + + return cfg; + } + /** * @param startGrid Start grid flag. */ http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/1652fd18/modules/core/src/test/java/org/apache/ignite/internal/GridProjectionForCachesSelfTest.java ---------------------------------------------------------------------- diff --git a/modules/core/src/test/java/org/apache/ignite/internal/GridProjectionForCachesSelfTest.java b/modules/core/src/test/java/org/apache/ignite/internal/GridProjectionForCachesSelfTest.java index 0a961d4..04cd3f0 100644 --- a/modules/core/src/test/java/org/apache/ignite/internal/GridProjectionForCachesSelfTest.java +++ b/modules/core/src/test/java/org/apache/ignite/internal/GridProjectionForCachesSelfTest.java @@ -52,9 +52,14 @@ public class GridProjectionForCachesSelfTest extends GridCommonAbstractTest { cfg.setDiscoverySpi(discoverySpi()); - cfg.setCacheConfiguration( - cacheConfiguration(null, new AttributeFilter(getTestGridName(0)), false), - cacheConfiguration(CACHE_NAME, new AttributeFilter(getTestGridName(2), getTestGridName(3)), true)); + List ccfgs = new ArrayList<>(); + + if (gridName.equals(getTestGridName(0))) + ccfgs.add(cacheConfiguration(null, new AttributeFilter(getTestGridName(0)), false)); + else if (gridName.equals(getTestGridName(2)) || gridName.equals(getTestGridName(3))) + ccfgs.add(cacheConfiguration(CACHE_NAME, new AttributeFilter(getTestGridName(2), getTestGridName(3)), true)); + + cfg.setCacheConfiguration(ccfgs.toArray(new CacheConfiguration[ccfgs.size()])); return cfg; } http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/1652fd18/modules/core/src/test/java/org/apache/ignite/internal/GridReleaseTypeSelfTest.java ---------------------------------------------------------------------- diff --git a/modules/core/src/test/java/org/apache/ignite/internal/GridReleaseTypeSelfTest.java b/modules/core/src/test/java/org/apache/ignite/internal/GridReleaseTypeSelfTest.java index 284aa0c..96d5f1a 100644 --- a/modules/core/src/test/java/org/apache/ignite/internal/GridReleaseTypeSelfTest.java +++ b/modules/core/src/test/java/org/apache/ignite/internal/GridReleaseTypeSelfTest.java @@ -27,7 +27,6 @@ import org.apache.ignite.testframework.junits.common.*; import java.io.*; import java.util.*; -import java.util.concurrent.atomic.*; /** * Test grids starting with non compatible release types. @@ -36,49 +35,56 @@ public class GridReleaseTypeSelfTest extends GridCommonAbstractTest { /** IP finder. */ private static final TcpDiscoveryIpFinder IP_FINDER = new TcpDiscoveryVmIpFinder(true); - /** Counter. */ - private static final AtomicInteger cnt = new AtomicInteger(); - /** */ - private String firstNodeVer; + private String nodeVer; /** */ - private String secondNodeVer; + private boolean clientMode; /** {@inheritDoc} */ @Override protected IgniteConfiguration getConfiguration(String gridName) throws Exception { IgniteConfiguration cfg = super.getConfiguration(gridName); - final int idx = cnt.getAndIncrement(); + if (clientMode) + cfg.setClientMode(true); - // Override node attributes in discovery spi. TcpDiscoverySpi discoSpi = new TcpDiscoverySpi() { - @Override public void setNodeAttributes(Map attrs, IgniteProductVersion ver) { + @Override public void setNodeAttributes(Map attrs, + IgniteProductVersion ver) { super.setNodeAttributes(attrs, ver); - if (idx % 2 == 0) - attrs.put(IgniteNodeAttributes.ATTR_BUILD_VER, firstNodeVer); - else - attrs.put(IgniteNodeAttributes.ATTR_BUILD_VER, secondNodeVer); + attrs.put(IgniteNodeAttributes.ATTR_BUILD_VER, nodeVer); } }; - discoSpi.setIpFinder(IP_FINDER); + discoSpi.setIpFinder(IP_FINDER).setForceServerMode(true); cfg.setDiscoverySpi(discoSpi); return cfg; } + /** {@inheritDoc} */ + @Override protected void afterTest() throws Exception { + clientMode = false; + + stopAllGrids(); + } + /** * @throws Exception If failed. */ public void testOsEditionDoesNotSupportRollingUpdates() throws Exception { - firstNodeVer = "1.0.0"; - secondNodeVer = "1.0.1"; + nodeVer = "1.0.0"; + + startGrid(0); try { - startGrids(2); + nodeVer = "1.0.1"; + + startGrid(1); + + fail("Exception has not been thrown."); } catch (IgniteCheckedException e) { StringWriter errors = new StringWriter(); @@ -87,17 +93,36 @@ public class GridReleaseTypeSelfTest extends GridCommonAbstractTest { String stackTrace = errors.toString(); - assertTrue( - "Caught exception does not contain specified string.", - stackTrace.contains("Local node and remote node have different version numbers") - ); - - return; + if (!stackTrace.contains("Local node and remote node have different version numbers")) + throw e; } - finally { - stopAllGrids(); + } + + /** + * @throws Exception If failed. + */ + public void testOsEditionDoesNotSupportRollingUpdatesClientMode() throws Exception { + nodeVer = "1.0.0"; + + startGrid(0); + + try { + nodeVer = "1.0.1"; + clientMode = true; + + startGrid(1); + + fail("Exception has not been thrown."); } + catch (IgniteCheckedException e) { + StringWriter errors = new StringWriter(); + + e.printStackTrace(new PrintWriter(errors)); - fail("Exception has not been thrown."); + String stackTrace = errors.toString(); + + if (!stackTrace.contains("Local node and remote node have different version numbers")) + throw e; + } } } http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/1652fd18/modules/core/src/test/java/org/apache/ignite/internal/GridSelfTest.java ---------------------------------------------------------------------- diff --git a/modules/core/src/test/java/org/apache/ignite/internal/GridSelfTest.java b/modules/core/src/test/java/org/apache/ignite/internal/GridSelfTest.java index b4dce6c..7f5ee54 100644 --- a/modules/core/src/test/java/org/apache/ignite/internal/GridSelfTest.java +++ b/modules/core/src/test/java/org/apache/ignite/internal/GridSelfTest.java @@ -118,8 +118,8 @@ public class GridSelfTest extends GridProjectionAbstractTest { g.message().remoteListen(null, new MessagingListenActor() { @Override protected void receive(UUID nodeId, String rcvMsg) throws Throwable { - assert locNodeId.equals(nodeId); - assert msg.equals(rcvMsg); + assertEquals(locNodeId, nodeId); + assertEquals(msg, rcvMsg); stop(rcvMsg); } http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/1652fd18/modules/core/src/test/java/org/apache/ignite/internal/managers/discovery/GridDiscoveryManagerAliveCacheSelfTest.java ---------------------------------------------------------------------- diff --git a/modules/core/src/test/java/org/apache/ignite/internal/managers/discovery/GridDiscoveryManagerAliveCacheSelfTest.java b/modules/core/src/test/java/org/apache/ignite/internal/managers/discovery/GridDiscoveryManagerAliveCacheSelfTest.java index b36c6f1..f2afb07 100644 --- a/modules/core/src/test/java/org/apache/ignite/internal/managers/discovery/GridDiscoveryManagerAliveCacheSelfTest.java +++ b/modules/core/src/test/java/org/apache/ignite/internal/managers/discovery/GridDiscoveryManagerAliveCacheSelfTest.java @@ -52,7 +52,7 @@ public class GridDiscoveryManagerAliveCacheSelfTest extends GridCommonAbstractTe private static final int TMP_NODES_CNT = 3; /** */ - private static final int ITERATIONS = 20; + private static final int ITERATIONS = 10; /** */ private int gridCntr; @@ -64,6 +64,9 @@ public class GridDiscoveryManagerAliveCacheSelfTest extends GridCommonAbstractTe private volatile CountDownLatch latch; /** */ + private boolean clientMode; + + /** */ private final IgnitePredicate lsnr = new IgnitePredicate() { @Override public boolean apply(Event evt) { assertNotNull("Topology lost nodes before stopTempNodes() was called.", latch); @@ -88,7 +91,15 @@ public class GridDiscoveryManagerAliveCacheSelfTest extends GridCommonAbstractTe TcpDiscoverySpi disc = new TcpDiscoverySpi(); + if (clientMode && ((gridName.charAt(gridName.length() - 1) - '0') & 1) != 0) + cfg.setClientMode(true); + else + disc.setMaxMissedClientHeartbeats(50); + + disc.setHeartbeatFrequency(500); disc.setIpFinder(IP_FINDER); + disc.setAckTimeout(1000); + disc.setSocketTimeout(1000); cfg.setCacheConfiguration(cCfg); cfg.setDiscoverySpi(disc); @@ -118,7 +129,7 @@ public class GridDiscoveryManagerAliveCacheSelfTest extends GridCommonAbstractTe /** * @throws Exception If failed. */ - public void testAlives() throws Exception { + private void doTestAlive() throws Exception { for (int i = 0; i < ITERATIONS; i++) { info("Performing iteration: " + i); @@ -141,6 +152,24 @@ public class GridDiscoveryManagerAliveCacheSelfTest extends GridCommonAbstractTe } /** + * @throws Exception If failed. + */ + public void testAlives() throws Exception { + clientMode = false; + + doTestAlive(); + } + + /** + * @throws Exception If failed. + */ + public void testAlivesClient() throws Exception { + clientMode = true; + + doTestAlive(); + } + + /** * Waits while topology on all nodes became equals to the expected size. * * @param nodesCnt Expected nodes count. @@ -149,6 +178,8 @@ public class GridDiscoveryManagerAliveCacheSelfTest extends GridCommonAbstractTe @SuppressWarnings("BusyWait") private void awaitDiscovery(long nodesCnt) throws InterruptedException { for (Ignite g : alive) { + ((TcpDiscoverySpi)g.configuration().getDiscoverySpi()).waitForClientMessagePrecessed(); + while (g.cluster().nodes().size() != nodesCnt) Thread.sleep(10); } @@ -187,7 +218,7 @@ public class GridDiscoveryManagerAliveCacheSelfTest extends GridCommonAbstractTe }); assertTrue( - currTop.contains(GridCacheUtils.oldest(k.internalCache().context(), new AffinityTopologyVersion(currVer)))); + currTop.contains(GridCacheUtils.oldestAliveCacheServerNode(k.context().cache().context(), new AffinityTopologyVersion(currVer)))); } } } @@ -213,23 +244,28 @@ public class GridDiscoveryManagerAliveCacheSelfTest extends GridCommonAbstractTe * Stops temporary nodes. */ private void stopTempNodes() { - int rmv = 0; + Collection toRmv = new ArrayList<>(alive.subList(0, TMP_NODES_CNT)); - Collection toRmv = new ArrayList<>(TMP_NODES_CNT); + alive.removeAll(toRmv); + + // Remove listeners to avoid receiving events from stopping nodes. + for (Ignite g : toRmv) + g.events().stopLocalListen(lsnr, EventType.EVT_NODE_LEFT, EventType.EVT_NODE_FAILED); - for (Iterator iter = alive.iterator(); iter.hasNext() && rmv < TMP_NODES_CNT;) { - toRmv.add(iter.next()); + for (Iterator itr = toRmv.iterator(); itr.hasNext(); ) { + Ignite g = itr.next(); - iter.remove(); + if (g.cluster().localNode().isClient()) { + G.stop(g.name(), false); - rmv++; + itr.remove(); + } } - // Remove listeners to avoid receiving events from stopping nodes. - for (Ignite g : toRmv) - g.events().stopLocalListen(lsnr, EventType.EVT_NODE_LEFT, EventType.EVT_NODE_FAILED); + for (Ignite g : toRmv) { + assert !g.cluster().localNode().isClient(); - for (Ignite g : toRmv) G.stop(g.name(), false); + } } } http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/1652fd18/modules/core/src/test/java/org/apache/ignite/internal/managers/discovery/GridDiscoveryManagerAttributesSelfTest.java ---------------------------------------------------------------------- diff --git a/modules/core/src/test/java/org/apache/ignite/internal/managers/discovery/GridDiscoveryManagerAttributesSelfTest.java b/modules/core/src/test/java/org/apache/ignite/internal/managers/discovery/GridDiscoveryManagerAttributesSelfTest.java index 538ea39..e76c615 100644 --- a/modules/core/src/test/java/org/apache/ignite/internal/managers/discovery/GridDiscoveryManagerAttributesSelfTest.java +++ b/modules/core/src/test/java/org/apache/ignite/internal/managers/discovery/GridDiscoveryManagerAttributesSelfTest.java @@ -29,7 +29,7 @@ import static org.apache.ignite.configuration.DeploymentMode.*; /** * Tests for node attributes consistency checks. */ -public class GridDiscoveryManagerAttributesSelfTest extends GridCommonAbstractTest { +public abstract class GridDiscoveryManagerAttributesSelfTest extends GridCommonAbstractTest { /** */ private static final String PREFER_IPV4 = "java.net.preferIPv4Stack"; @@ -37,7 +37,7 @@ public class GridDiscoveryManagerAttributesSelfTest extends GridCommonAbstractTe private static final TcpDiscoveryIpFinder IP_FINDER = new TcpDiscoveryVmIpFinder(true); /** */ - private static DeploymentMode mode = SHARED; + private static DeploymentMode mode; /** */ private static boolean p2pEnabled; @@ -46,18 +46,34 @@ public class GridDiscoveryManagerAttributesSelfTest extends GridCommonAbstractTe @Override protected IgniteConfiguration getConfiguration(String gridName) throws Exception { IgniteConfiguration cfg = super.getConfiguration(gridName); - TcpDiscoverySpi disc = new TcpDiscoverySpi(); - - disc.setIpFinder(IP_FINDER); + if (gridName.equals(getTestGridName(1))) + cfg.setClientMode(true); cfg.setIncludeProperties(PREFER_IPV4); cfg.setDeploymentMode(mode); cfg.setPeerClassLoadingEnabled(p2pEnabled); - cfg.setDiscoverySpi(disc); + + TcpDiscoverySpi discoverySpi = new TcpDiscoverySpi(); + + discoverySpi.setIpFinder(IP_FINDER); + + cfg.setDiscoverySpi(discoverySpi); return cfg; } + /** {@inheritDoc} */ + @Override protected void beforeTest() throws Exception { + mode = SHARED; + + p2pEnabled = false; + } + + /** {@inheritDoc} */ + @Override protected void afterTest() throws Exception { + stopAllGrids(); + } + /** * @throws Exception If failed. */ @@ -83,44 +99,35 @@ public class GridDiscoveryManagerAttributesSelfTest extends GridCommonAbstractTe * @throws Exception If failed. */ public void testPreferIpV4StackDifferentValues() throws Exception { - try { - System.setProperty(PREFER_IPV4, "true"); + System.setProperty(PREFER_IPV4, "true"); - for (int i = 0; i < 2; i++) { - Ignite g = startGrid(i); + for (int i = 0; i < 2; i++) { + Ignite g = startGrid(i); - assert "true".equals(g.cluster().localNode().attribute(PREFER_IPV4)); - } + assert "true".equals(g.cluster().localNode().attribute(PREFER_IPV4)); + } - System.setProperty(PREFER_IPV4, "false"); + System.setProperty(PREFER_IPV4, "false"); - startGrid(2); - } - finally { - stopAllGrids(); - } + startGrid(2); } /** * @throws Exception If failed. */ public void testDifferentDeploymentModes() throws Exception { - try { - startGrid(1); + startGrid(0); - mode = CONTINUOUS; + mode = CONTINUOUS; - try { - startGrid(2); + try { + startGrid(1); - fail(); - } - catch (IgniteCheckedException e) { - assertTrue(e.getCause().getMessage().startsWith("Remote node has deployment mode different from")); - } + fail(); } - finally { - stopAllGrids(); + catch (IgniteCheckedException e) { + if (!e.getCause().getMessage().startsWith("Remote node has deployment mode different from")) + throw e; } } @@ -128,23 +135,18 @@ public class GridDiscoveryManagerAttributesSelfTest extends GridCommonAbstractTe * @throws Exception If failed. */ public void testDifferentPeerClassLoadingEnabledFlag() throws Exception { - try { - startGrid(1); + startGrid(0); - p2pEnabled = true; + p2pEnabled = true; - try { - startGrid(2); + try { + startGrid(1); - fail(); - } - catch (IgniteCheckedException e) { - assertTrue(e.getCause().getMessage().startsWith("Remote node has peer class loading enabled flag " + - "different from")); - } + fail(); } - finally { - stopAllGrids(); + catch (IgniteCheckedException e) { + if (!e.getCause().getMessage().startsWith("Remote node has peer class loading enabled flag different from")) + throw e; } } @@ -153,19 +155,35 @@ public class GridDiscoveryManagerAttributesSelfTest extends GridCommonAbstractTe * @throws Exception If failed. */ private void testPreferIpV4Stack(boolean preferIpV4) throws Exception { - try { - String val = String.valueOf(preferIpV4); + String val = String.valueOf(preferIpV4); - System.setProperty(PREFER_IPV4, val); + System.setProperty(PREFER_IPV4, val); - for (int i = 0; i < 2; i++) { - Ignite g = startGrid(i); + for (int i = 0; i < 2; i++) { + Ignite g = startGrid(i); - assert val.equals(g.cluster().localNode().attribute(PREFER_IPV4)); - } + assert val.equals(g.cluster().localNode().attribute(PREFER_IPV4)); } - finally { - stopAllGrids(); + } + + /** + * + */ + public static class RegularDiscovery extends GridDiscoveryManagerAttributesSelfTest { + /** {@inheritDoc} */ + @Override protected IgniteConfiguration getConfiguration(String gridName) throws Exception { + IgniteConfiguration cfg = super.getConfiguration(gridName); + + ((TcpDiscoverySpi)cfg.getDiscoverySpi()).setForceServerMode(true); + + return cfg; } } + + /** + * + */ + public static class ClientDiscovery extends GridDiscoveryManagerAttributesSelfTest { + // No-op. + } } http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/1652fd18/modules/core/src/test/java/org/apache/ignite/internal/managers/discovery/GridDiscoveryManagerSelfTest.java ---------------------------------------------------------------------- diff --git a/modules/core/src/test/java/org/apache/ignite/internal/managers/discovery/GridDiscoveryManagerSelfTest.java b/modules/core/src/test/java/org/apache/ignite/internal/managers/discovery/GridDiscoveryManagerSelfTest.java index 6f7c935..65aec49 100644 --- a/modules/core/src/test/java/org/apache/ignite/internal/managers/discovery/GridDiscoveryManagerSelfTest.java +++ b/modules/core/src/test/java/org/apache/ignite/internal/managers/discovery/GridDiscoveryManagerSelfTest.java @@ -33,7 +33,7 @@ import static org.apache.ignite.cache.CacheMode.*; /** * */ -public class GridDiscoveryManagerSelfTest extends GridCommonAbstractTest { +public abstract class GridDiscoveryManagerSelfTest extends GridCommonAbstractTest { /** */ private static final String CACHE_NAME = "cache"; @@ -50,12 +50,6 @@ public class GridDiscoveryManagerSelfTest extends GridCommonAbstractTest { @Override protected IgniteConfiguration getConfiguration(String gridName) throws Exception { IgniteConfiguration cfg = super.getConfiguration(gridName); - TcpDiscoverySpi disc = new TcpDiscoverySpi(); - - disc.setIpFinder(IP_FINDER); - - cfg.setDiscoverySpi(disc); - CacheConfiguration ccfg1 = defaultCacheConfiguration(); ccfg1.setName(CACHE_NAME); @@ -66,14 +60,21 @@ public class GridDiscoveryManagerSelfTest extends GridCommonAbstractTest { if (gridName.equals(getTestGridName(1))) cfg.setClientMode(true); + else { + ccfg1.setNearConfiguration(null); + ccfg2.setNearConfiguration(null); - ccfg1.setNearConfiguration(null); - ccfg2.setNearConfiguration(null); + ccfg1.setCacheMode(PARTITIONED); + ccfg2.setCacheMode(PARTITIONED); + + cfg.setCacheConfiguration(ccfg1, ccfg2); + } - ccfg1.setCacheMode(PARTITIONED); - ccfg2.setCacheMode(PARTITIONED); + TcpDiscoverySpi discoverySpi = new TcpDiscoverySpi(); - cfg.setCacheConfiguration(ccfg1, ccfg2); + discoverySpi.setIpFinder(IP_FINDER); + + cfg.setDiscoverySpi(discoverySpi); return cfg; } @@ -186,4 +187,25 @@ public class GridDiscoveryManagerSelfTest extends GridCommonAbstractTest { assertTrue(g0.context().discovery().hasNearCache(null, four)); assertFalse(g0.context().discovery().hasNearCache(null, five)); } + + /** + * + */ + public static class RegularDiscovery extends GridDiscoveryManagerSelfTest { + /** {@inheritDoc} */ + @Override protected IgniteConfiguration getConfiguration(String gridName) throws Exception { + IgniteConfiguration cfg = super.getConfiguration(gridName); + + ((TcpDiscoverySpi)cfg.getDiscoverySpi()).setForceServerMode(true); + + return cfg; + } + } + + /** + * + */ + public static class ClientDiscovery extends GridDiscoveryManagerSelfTest { + // No-op. + } } http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/1652fd18/modules/core/src/test/java/org/apache/ignite/internal/processors/affinity/GridAffinityProcessorAbstractSelfTest.java ---------------------------------------------------------------------- diff --git a/modules/core/src/test/java/org/apache/ignite/internal/processors/affinity/GridAffinityProcessorAbstractSelfTest.java b/modules/core/src/test/java/org/apache/ignite/internal/processors/affinity/GridAffinityProcessorAbstractSelfTest.java index e7fab8e..7da3728 100644 --- a/modules/core/src/test/java/org/apache/ignite/internal/processors/affinity/GridAffinityProcessorAbstractSelfTest.java +++ b/modules/core/src/test/java/org/apache/ignite/internal/processors/affinity/GridAffinityProcessorAbstractSelfTest.java @@ -54,6 +54,7 @@ public abstract class GridAffinityProcessorAbstractSelfTest extends GridCommonAb TcpDiscoverySpi discoSpi = new TcpDiscoverySpi(); + discoSpi.setForceServerMode(true); discoSpi.setIpFinder(ipFinder); cfg.setDiscoverySpi(discoSpi); http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/1652fd18/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/CacheReadOnlyTransactionalClientSelfTest.java ---------------------------------------------------------------------- diff --git a/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/CacheReadOnlyTransactionalClientSelfTest.java b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/CacheReadOnlyTransactionalClientSelfTest.java new file mode 100644 index 0000000..f2c38e1 --- /dev/null +++ b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/CacheReadOnlyTransactionalClientSelfTest.java @@ -0,0 +1,327 @@ +/* + * 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.cache.*; +import org.apache.ignite.cache.store.*; +import org.apache.ignite.configuration.*; +import org.apache.ignite.internal.util.typedef.*; +import org.apache.ignite.spi.discovery.tcp.*; +import org.apache.ignite.spi.discovery.tcp.ipfinder.*; +import org.apache.ignite.spi.discovery.tcp.ipfinder.vm.*; +import org.apache.ignite.testframework.junits.common.*; + +import javax.cache.*; +import javax.cache.configuration.*; +import javax.cache.processor.*; + +import static org.apache.ignite.IgniteSystemProperties.*; + +/** + * Tests for read-only transactional cache client. + */ +public class CacheReadOnlyTransactionalClientSelfTest extends GridCommonAbstractTest { + /** */ + private static final TcpDiscoveryIpFinder IP_FINDER = new TcpDiscoveryVmIpFinder(true); + + /** */ + private static final String CACHE_NAME = "test-cache"; + + /** */ + private boolean client; + + /** */ + private boolean nearEnabled; + + /** */ + private Factory factory; + + /** {@inheritDoc} */ + @Override protected IgniteConfiguration getConfiguration(String gridName) throws Exception { + IgniteConfiguration cfg = super.getConfiguration(gridName); + + cfg.setClientMode(client); + + CacheConfiguration cc = new CacheConfiguration(); + + cc.setName(CACHE_NAME); + cc.setAtomicityMode(CacheAtomicityMode.TRANSACTIONAL); + cc.setCacheStoreFactory(factory); + + if (client && nearEnabled) + cc.setNearConfiguration(new NearCacheConfiguration()); + + cfg.setCacheConfiguration(cc); + + TcpDiscoverySpi disco = new TcpDiscoverySpi(); + + disco.setIpFinder(IP_FINDER); + + cfg.setDiscoverySpi(disco); + + return cfg; + } + + /** {@inheritDoc} */ + @Override protected void beforeTestsStarted() throws Exception { + client = false; + factory = new Factory1(); + + startGrids(2); + } + + /** {@inheritDoc} */ + @Override protected void afterTestsStopped() throws Exception { + stopAllGrids(); + } + + /** {@inheritDoc} */ + @Override protected void afterTest() throws Exception { + stopGrid(); + } + + /** + * @throws Exception If failed. + */ + public void testCorrectStore() throws Exception { + client = true; + nearEnabled = false; + factory = new Factory1(); + + Ignite ignite = startGrid(); + + IgniteCache cache = ignite.cache(CACHE_NAME); + + cache.get(0); + cache.getAll(F.asSet(0, 1)); + cache.getAndPut(0, 0); + cache.getAndPutIfAbsent(0, 0); + cache.getAndRemove(0); + cache.getAndReplace(0, 0); + cache.put(0, 0); + cache.putAll(F.asMap(0, 0, 1, 1)); + cache.putIfAbsent(0, 0); + cache.remove(0); + cache.remove(0, 0); + cache.removeAll(F.asSet(0, 1)); + cache.removeAll(); + cache.invoke(0, new EP()); + cache.invokeAll(F.asSet(0, 1), new EP()); + } + + /** + * @throws Exception If failed. + */ + public void testInvalidStore() throws Exception { + client = true; + nearEnabled = false; + factory = new Factory2(); + + try { + startGrid(); + + assert false : "Exception was not thrown."; + } + catch (Exception e) { + assert e.getMessage().startsWith("Store factory mismatch") : e.getMessage(); + } + } + + /** + * @throws Exception If failed. + */ + public void testDisabledConsistencyCheck() throws Exception { + client = false; + nearEnabled = false; + factory = new Factory2(); + + System.setProperty(IGNITE_SKIP_CONFIGURATION_CONSISTENCY_CHECK, "true"); + + startGrid("client-1"); + + factory = new Factory1(); + + System.clearProperty(IGNITE_SKIP_CONFIGURATION_CONSISTENCY_CHECK); + + startGrid("client-2"); + } + + /** + * @throws Exception If failed. + */ + public void testNoStoreNearDisabled() throws Exception { + nearEnabled = false; + + doTestNoStore(); + } + + /** + * @throws Exception If failed. + */ + public void testNoStoreNearEnabled() throws Exception { + nearEnabled = true; + + doTestNoStore(); + } + + /** + * @throws Exception If failed. + */ + private void doTestNoStore() throws Exception { + client = true; + factory = null; + + Ignite ignite = startGrid(); + + IgniteCache cache = ignite.cache(CACHE_NAME); + + cache.get(0); + cache.getAll(F.asSet(0, 1)); + + try { + cache.getAndPut(0, 0); + } + catch (CacheException e) { + assert e.getMessage().startsWith("Updates are not allowed for transactional cache: " + CACHE_NAME + ".") : + e.getMessage(); + } + + try { + cache.getAndPutIfAbsent(0, 0); + } + catch (CacheException e) { + assert e.getMessage().startsWith("Updates are not allowed for transactional cache: " + CACHE_NAME + ".") : + e.getMessage(); + } + + try { + cache.getAndRemove(0); + } + catch (CacheException e) { + assert e.getMessage().startsWith("Updates are not allowed for transactional cache: " + CACHE_NAME + ".") : + e.getMessage(); + } + + try { + cache.getAndReplace(0, 0); + } + catch (CacheException e) { + assert e.getMessage().startsWith("Updates are not allowed for transactional cache: " + CACHE_NAME + ".") : + e.getMessage(); + } + + try { + cache.put(0, 0); + } + catch (CacheException e) { + assert e.getMessage().startsWith("Updates are not allowed for transactional cache: " + CACHE_NAME + ".") : + e.getMessage(); + } + + try { + cache.putAll(F.asMap(0, 0, 1, 1)); + } + catch (CacheException e) { + assert e.getMessage().startsWith("Updates are not allowed for transactional cache: " + CACHE_NAME + ".") : + e.getMessage(); + } + + try { + cache.putIfAbsent(0, 0); + } + catch (CacheException e) { + assert e.getMessage().startsWith("Updates are not allowed for transactional cache: " + CACHE_NAME + ".") : + e.getMessage(); + } + + try { + cache.remove(0); + } + catch (CacheException e) { + assert e.getMessage().startsWith("Updates are not allowed for transactional cache: " + CACHE_NAME + ".") : + e.getMessage(); + } + + try { + cache.remove(0, 0); + } + catch (CacheException e) { + assert e.getMessage().startsWith("Updates are not allowed for transactional cache: " + CACHE_NAME + ".") : + e.getMessage(); + } + + try { + cache.removeAll(F.asSet(0, 1)); + } + catch (CacheException e) { + assert e.getMessage().startsWith("Updates are not allowed for transactional cache: " + CACHE_NAME + ".") : + e.getMessage(); + } + + try { + cache.removeAll(); + } + catch (CacheException e) { + assert e.getMessage().startsWith("Updates are not allowed for transactional cache: " + CACHE_NAME + ".") : + e.getMessage(); + } + + try { + cache.invoke(0, new EP()); + } + catch (CacheException e) { + assert e.getMessage().startsWith("Updates are not allowed for transactional cache: " + CACHE_NAME + ".") : + e.getMessage(); + } + + try { + cache.invokeAll(F.asSet(0, 1), new EP()); + } + catch (CacheException e) { + assert e.getMessage().startsWith("Updates are not allowed for transactional cache: " + CACHE_NAME + ".") : + e.getMessage(); + } + } + + /** + */ + private static class Factory1 implements Factory { + /** {@inheritDoc} */ + @Override public CacheStore create() { + return null; + } + } + + /** + */ + private static class Factory2 implements Factory { + /** {@inheritDoc} */ + @Override public CacheStore create() { + return null; + } + } + + /** + */ + private static class EP implements CacheEntryProcessor { + @Override public Object process(MutableEntry entry, Object... arguments) { + return null; + } + } +} http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/1652fd18/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/CacheRemoveAllSelfTest.java ---------------------------------------------------------------------- diff --git a/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/CacheRemoveAllSelfTest.java b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/CacheRemoveAllSelfTest.java index f5de96f..1d4d2f4 100644 --- a/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/CacheRemoveAllSelfTest.java +++ b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/CacheRemoveAllSelfTest.java @@ -32,7 +32,7 @@ import java.util.concurrent.atomic.*; public class CacheRemoveAllSelfTest extends GridCacheAbstractSelfTest { /** {@inheritDoc} */ @Override protected long getTestTimeout() { - return 60000; + return 2 * 60 * 1000; } /** {@inheritDoc} */ http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/1652fd18/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/GridCacheAbstractFailoverSelfTest.java ---------------------------------------------------------------------- diff --git a/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/GridCacheAbstractFailoverSelfTest.java b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/GridCacheAbstractFailoverSelfTest.java index 5d9ad35..6b7d1b8 100644 --- a/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/GridCacheAbstractFailoverSelfTest.java +++ b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/GridCacheAbstractFailoverSelfTest.java @@ -74,7 +74,7 @@ public abstract class GridCacheAbstractFailoverSelfTest extends GridCacheAbstrac discoSpi.setSocketTimeout(10_000); discoSpi.setAckTimeout(10_000); - discoSpi.setNetworkTimeout(10_000); + discoSpi.setNetworkTimeout(60_000); return cfg; } http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/1652fd18/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/GridCacheAbstractFullApiSelfTest.java ---------------------------------------------------------------------- diff --git a/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/GridCacheAbstractFullApiSelfTest.java b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/GridCacheAbstractFullApiSelfTest.java index 70d8f9c..9bfbd15 100644 --- a/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/GridCacheAbstractFullApiSelfTest.java +++ b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/GridCacheAbstractFullApiSelfTest.java @@ -33,6 +33,7 @@ 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.lang.*; +import org.apache.ignite.spi.discovery.tcp.*; import org.apache.ignite.spi.swapspace.inmemory.*; import org.apache.ignite.testframework.*; import org.apache.ignite.transactions.*; @@ -130,6 +131,8 @@ public abstract class GridCacheAbstractFullApiSelfTest extends GridCacheAbstract @Override protected IgniteConfiguration getConfiguration(String gridName) throws Exception { IgniteConfiguration cfg = super.getConfiguration(gridName); + ((TcpDiscoverySpi)cfg.getDiscoverySpi()).setForceServerMode(true); + if (memoryMode() == OFFHEAP_TIERED || memoryMode() == OFFHEAP_VALUES) cfg.setSwapSpaceSpi(new GridTestSwapSpaceSpi()); @@ -3913,6 +3916,33 @@ public abstract class GridCacheAbstractFullApiSelfTest extends GridCacheAbstract /** * @throws Exception If failed. */ + public void testIterator() throws Exception { + IgniteCache cache = grid(0).cache(null); + + final int KEYS = 1000; + + for (int i = 0; i < KEYS; i++) + cache.put(i, i); + + // Try to initialize readers in case when near cache is enabled. + for (int i = 0; i < gridCount(); i++) { + cache = grid(i).cache(null); + + for (int k = 0; k < KEYS; k++) + assertEquals((Object)k, cache.get(k)); + } + + int cnt = 0; + + for (Cache.Entry e : cache) + cnt++; + + assertEquals(KEYS, cnt); + } + + /** + * @throws Exception If failed. + */ public void testIgniteCacheIterator() throws Exception { IgniteCache cache = jcache(0); @@ -4863,6 +4893,89 @@ public abstract class GridCacheAbstractFullApiSelfTest extends GridCacheAbstract } /** + * @throws Exception If failed. + */ + public void testGetOutTx() throws Exception { + checkGetOutTx(false); + } + + /** + * @throws Exception If failed. + */ + public void testGetOutTxAsync() throws Exception { + checkGetOutTx(true); + } + + /** + * @throws Exception If failed. + */ + private void checkGetOutTx(boolean async) throws Exception { + final AtomicInteger lockEvtCnt = new AtomicInteger(); + + IgnitePredicate lsnr = new IgnitePredicate() { + @Override public boolean apply(Event evt) { + lockEvtCnt.incrementAndGet(); + + return true; + } + }; + + try { + IgniteCache cache = grid(0).cache(null); + + List keys = primaryKeysForCache(cache, 2); + + assertEquals(2, keys.size()); + + cache.put(keys.get(0), 0); + cache.put(keys.get(1), 1); + + grid(0).events().localListen(lsnr, EVT_CACHE_OBJECT_LOCKED, EVT_CACHE_OBJECT_UNLOCKED); + + if (async) + cache = cache.withAsync(); + + try (Transaction tx = transactions().txStart(PESSIMISTIC, REPEATABLE_READ)) { + Integer val0 = cache.get(keys.get(0)); + + if (async) + val0 = cache.future().get(); + + assertEquals(0, val0.intValue()); + + Map allOutTx = cache.getAllOutTx(F.asSet(keys.get(1))); + + if (async) + allOutTx = cache.>future().get(); + + assertEquals(1, allOutTx.size()); + + assertTrue(allOutTx.containsKey(keys.get(1))); + + assertEquals(1, allOutTx.get(keys.get(1)).intValue()); + } + + assertTrue(GridTestUtils.waitForCondition(new PA() { + @Override public boolean apply() { + info("Lock event count: " + lockEvtCnt.get()); + if (atomicityMode() == ATOMIC) + return lockEvtCnt.get() == 0; + + if (cacheMode() == PARTITIONED && nearEnabled()) { + if (!grid(0).configuration().isClientMode()) + return lockEvtCnt.get() == 4; + } + + return lockEvtCnt.get() == 2; + } + }, 15000)); + } + finally { + grid(0).events().stopLocalListen(lsnr, EVT_CACHE_OBJECT_LOCKED, EVT_CACHE_OBJECT_UNLOCKED); + } + } + + /** * Sets given value, returns old value. */ public static final class SetValueProcessor implements EntryProcessor { http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/1652fd18/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/GridCacheAbstractMetricsSelfTest.java ---------------------------------------------------------------------- diff --git a/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/GridCacheAbstractMetricsSelfTest.java b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/GridCacheAbstractMetricsSelfTest.java index 1821e12..bb1732e 100644 --- a/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/GridCacheAbstractMetricsSelfTest.java +++ b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/GridCacheAbstractMetricsSelfTest.java @@ -38,7 +38,7 @@ import static java.util.concurrent.TimeUnit.*; */ public abstract class GridCacheAbstractMetricsSelfTest extends GridCacheAbstractSelfTest { /** */ - private static final int KEY_CNT = 50; + private static final int KEY_CNT = 500; /** {@inheritDoc} */ @Override protected boolean swapEnabled() { @@ -163,24 +163,18 @@ public abstract class GridCacheAbstractMetricsSelfTest extends GridCacheAbstract IgniteCache cacheAsync = cache.withAsync(); - cache.put(1, 1); - cache.put(2, 2); + for (int i = 0; i < KEY_CNT; i++) + cache.put(i, i); assertEquals(cache.metrics().getAverageRemoveTime(), 0.0, 0.0); - cacheAsync.getAndRemove(1); - - IgniteFuture fut = cacheAsync.future(); - - assertEquals(1, (int)fut.get()); - - assert cache.metrics().getAverageRemoveTime() > 0; - - cacheAsync.getAndRemove(2); + for (int i = 0; i < KEY_CNT; i++) { + cacheAsync.getAndRemove(i); - fut = cacheAsync.future(); + IgniteFuture fut = cacheAsync.future(); - assertEquals(2, (int)fut.get()); + fut.get(); + } assert cache.metrics().getAverageRemoveTime() > 0; } @@ -221,18 +215,13 @@ public abstract class GridCacheAbstractMetricsSelfTest extends GridCacheAbstract public void testRemoveAvgTime() throws Exception { IgniteCache cache = grid(0).cache(null); - cache.put(1, 1); - cache.put(2, 2); + for (int i = 0; i < KEY_CNT; i++) + cache.put(i, i); assertEquals(cache.metrics().getAverageRemoveTime(), 0.0, 0.0); - cache.remove(1); - - float avgRmvTime = cache.metrics().getAverageRemoveTime(); - - assert avgRmvTime > 0; - - cache.remove(2); + for (int i = 0; i < KEY_CNT; i++) + cache.remove(i); assert cache.metrics().getAverageRemoveTime() > 0; } @@ -378,17 +367,12 @@ public abstract class GridCacheAbstractMetricsSelfTest extends GridCacheAbstract assertEquals(0.0, cache.metrics().getAveragePutTime(), 0.0); assertEquals(0, cache.metrics().getCachePuts()); - cache.put(1, 1); - - float avgPutTime = cache.metrics().getAveragePutTime(); - - assert avgPutTime >= 0; - - assertEquals(1, cache.metrics().getCachePuts()); + for (int i = 0; i < KEY_CNT; i++) + cache.put(i, i); - cache.put(2, 2); + assert cache.metrics().getAveragePutTime() > 0; - assert cache.metrics().getAveragePutTime() >= 0; + assertEquals(KEY_CNT, cache.metrics().getCachePuts()); } /** http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/1652fd18/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/GridCacheAbstractRemoveFailureTest.java ---------------------------------------------------------------------- diff --git a/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/GridCacheAbstractRemoveFailureTest.java b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/GridCacheAbstractRemoveFailureTest.java index c6ede61..2b6a6b0 100644 --- a/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/GridCacheAbstractRemoveFailureTest.java +++ b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/GridCacheAbstractRemoveFailureTest.java @@ -18,10 +18,12 @@ package org.apache.ignite.internal.processors.cache; import org.apache.ignite.*; +import org.apache.ignite.configuration.*; import org.apache.ignite.internal.*; 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.tcp.*; import org.apache.ignite.testframework.*; import org.jsr166.*; @@ -71,6 +73,18 @@ public abstract class GridCacheAbstractRemoveFailureTest extends GridCacheAbstra private String sizePropVal; /** {@inheritDoc} */ + @Override protected IgniteConfiguration getConfiguration(String gridName) throws Exception { + IgniteConfiguration cfg = super.getConfiguration(gridName); + + ((TcpDiscoverySpi)cfg.getDiscoverySpi()).setForceServerMode(true); + + if (testClientNode() && getTestGridName(0).equals(gridName)) + cfg.setClientMode(true); + + return cfg; + } + + /** {@inheritDoc} */ @Override protected int gridCount() { return GRID_CNT; } @@ -106,9 +120,18 @@ public abstract class GridCacheAbstractRemoveFailureTest extends GridCacheAbstra } /** + * @return {@code True} if test updates from client node. + */ + protected boolean testClientNode() { + return false; + } + + /** * @throws Exception If failed. */ public void testPutAndRemove() throws Exception { + assertEquals(testClientNode(), (boolean)grid(0).configuration().isClientMode()); + final IgniteCache sndCache0 = grid(0).cache(null); final AtomicBoolean stop = new AtomicBoolean(); http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/1652fd18/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/GridCacheAbstractSelfTest.java ---------------------------------------------------------------------- diff --git a/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/GridCacheAbstractSelfTest.java b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/GridCacheAbstractSelfTest.java index efd0185..eeb9f45 100644 --- a/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/GridCacheAbstractSelfTest.java +++ b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/GridCacheAbstractSelfTest.java @@ -58,7 +58,7 @@ public abstract class GridCacheAbstractSelfTest extends GridCommonAbstractTest { protected static final Map map = new ConcurrentHashMap8<>(); /** VM ip finder for TCP discovery. */ - private static TcpDiscoveryIpFinder ipFinder = new TcpDiscoveryVmIpFinder(true); + protected static TcpDiscoveryIpFinder ipFinder = new TcpDiscoveryVmIpFinder(true); /** * @return Grids count to start. http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/1652fd18/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/GridCacheAtomicMessageCountSelfTest.java ---------------------------------------------------------------------- diff --git a/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/GridCacheAtomicMessageCountSelfTest.java b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/GridCacheAtomicMessageCountSelfTest.java index da2b81c..db4061a 100644 --- a/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/GridCacheAtomicMessageCountSelfTest.java +++ b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/GridCacheAtomicMessageCountSelfTest.java @@ -60,6 +60,7 @@ public class GridCacheAtomicMessageCountSelfTest extends GridCommonAbstractTest TcpDiscoverySpi discoSpi = new TcpDiscoverySpi(); + discoSpi.setForceServerMode(true); discoSpi.setIpFinder(ipFinder); cfg.setDiscoverySpi(discoSpi); http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/1652fd18/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/GridCacheConcurrentTxMultiNodeTest.java ---------------------------------------------------------------------- diff --git a/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/GridCacheConcurrentTxMultiNodeTest.java b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/GridCacheConcurrentTxMultiNodeTest.java index 8a1ae78..bba4ad9 100644 --- a/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/GridCacheConcurrentTxMultiNodeTest.java +++ b/modules/core/src/test/java/org/apache/ignite/internal/processors/cache/GridCacheConcurrentTxMultiNodeTest.java @@ -26,7 +26,6 @@ import org.apache.ignite.cluster.*; import org.apache.ignite.compute.*; import org.apache.ignite.configuration.*; import org.apache.ignite.internal.*; -import org.apache.ignite.internal.processors.cache.query.*; import org.apache.ignite.internal.processors.cache.distributed.dht.*; import org.apache.ignite.internal.processors.cache.distributed.near.*; import org.apache.ignite.internal.util.*; @@ -38,6 +37,7 @@ import org.apache.ignite.spi.discovery.tcp.ipfinder.*; import org.apache.ignite.spi.discovery.tcp.ipfinder.vm.*; import org.apache.ignite.testframework.junits.common.*; import org.apache.ignite.transactions.*; + import org.jetbrains.annotations.*; import java.io.*; @@ -110,7 +110,11 @@ public class GridCacheConcurrentTxMultiNodeTest extends GridCommonAbstractTest { CacheConfiguration cc = defaultCacheConfiguration(); cc.setCacheMode(mode); - cc.setEvictionPolicy(new LruEvictionPolicy(1000)); + + LruEvictionPolicy plc = new LruEvictionPolicy(); + plc.setMaxSize(1000); + + cc.setEvictionPolicy(plc); cc.setEvictSynchronized(false); cc.setSwapEnabled(false); cc.setWriteSynchronizationMode(FULL_SYNC);