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 058D517F0E for ; Thu, 9 Apr 2015 10:13:57 +0000 (UTC) Received: (qmail 32578 invoked by uid 500); 9 Apr 2015 10:13:56 -0000 Delivered-To: apmail-ignite-commits-archive@ignite.apache.org Received: (qmail 32544 invoked by uid 500); 9 Apr 2015 10:13:56 -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 32535 invoked by uid 99); 9 Apr 2015 10:13:56 -0000 Received: from nike.apache.org (HELO nike.apache.org) (192.87.106.230) by apache.org (qpsmtpd/0.29) with ESMTP; Thu, 09 Apr 2015 10:13:56 +0000 X-ASF-Spam-Status: No, hits=-2000.0 required=5.0 tests=ALL_TRUSTED,T_RP_MATCHES_RCVD,WEIRD_QUOTING 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; Thu, 09 Apr 2015 10:13:10 +0000 Received: (qmail 29741 invoked by uid 99); 9 Apr 2015 10:12:59 -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; Thu, 09 Apr 2015 10:12:59 +0000 Received: by git1-us-west.apache.org (ASF Mail Server at git1-us-west.apache.org, from userid 33) id AB9E7DFF8C; Thu, 9 Apr 2015 10:12:59 +0000 (UTC) Content-Type: text/plain; charset="us-ascii" MIME-Version: 1.0 Content-Transfer-Encoding: 8bit From: sboikov@apache.org To: commits@ignite.incubator.apache.org Date: Thu, 09 Apr 2015 10:13:37 -0000 Message-Id: In-Reply-To: References: X-Mailer: ASF-Git Admin Mailer Subject: [39/50] [abbrv] incubator-ignite git commit: ignite-635: IgniteCluster # startNodes return ClusterStartNodeResult-s instead of Tuple3-s X-Virus-Checked: Checked by ClamAV on apache.org ignite-635: IgniteCluster # startNodes return ClusterStartNodeResult-s instead of Tuple3-s Project: http://git-wip-us.apache.org/repos/asf/incubator-ignite/repo Commit: http://git-wip-us.apache.org/repos/asf/incubator-ignite/commit/7f4c5eed Tree: http://git-wip-us.apache.org/repos/asf/incubator-ignite/tree/7f4c5eed Diff: http://git-wip-us.apache.org/repos/asf/incubator-ignite/diff/7f4c5eed Branch: refs/heads/ignite-673 Commit: 7f4c5eed0af2b56b1dd671cbe76a4224b72f5045 Parents: 71439bc Author: Artem Shutak Authored: Wed Apr 8 18:27:23 2015 +0300 Committer: Artem Shutak Committed: Wed Apr 8 18:27:23 2015 +0300 ---------------------------------------------------------------------- .../java/org/apache/ignite/IgniteCluster.java | 9 +- .../ignite/cluster/ClusterStartNodeResult.java | 46 +++ .../cluster/ClusterStartNodeResultImpl.java | 87 ++++++ .../cluster/IgniteClusterAsyncImpl.java | 5 +- .../internal/cluster/IgniteClusterImpl.java | 37 +-- .../util/nodestart/IgniteNodeCallable.java | 30 -- .../util/nodestart/IgniteSshHelper.java | 2 +- .../util/nodestart/StartNodeCallable.java | 30 ++ .../util/nodestart/IgniteNodeCallableImpl.java | 308 ------------------ .../util/nodestart/IgniteSshHelperImpl.java | 4 +- .../util/nodestart/StartNodeCallableImpl.java | 309 +++++++++++++++++++ ...gniteProjectionStartStopRestartSelfTest.java | 261 ++++++++-------- .../IgniteStartStopRestartTestSuite.java | 1 + .../commands/start/VisorStartCommand.scala | 4 +- 14 files changed, 631 insertions(+), 502 deletions(-) ---------------------------------------------------------------------- http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/7f4c5eed/modules/core/src/main/java/org/apache/ignite/IgniteCluster.java ---------------------------------------------------------------------- diff --git a/modules/core/src/main/java/org/apache/ignite/IgniteCluster.java b/modules/core/src/main/java/org/apache/ignite/IgniteCluster.java index 7d7d2fb..02fea14 100644 --- a/modules/core/src/main/java/org/apache/ignite/IgniteCluster.java +++ b/modules/core/src/main/java/org/apache/ignite/IgniteCluster.java @@ -19,7 +19,6 @@ package org.apache.ignite; import org.apache.ignite.cache.affinity.*; import org.apache.ignite.cluster.*; -import org.apache.ignite.internal.util.lang.*; import org.apache.ignite.lang.*; import org.jetbrains.annotations.*; @@ -170,12 +169,12 @@ public interface IgniteCluster extends ClusterGroup, IgniteAsyncSupport { * nodes on the host than expected. * @param timeout Connection timeout. * @param maxConn Number of parallel SSH connections to one host. - * @return Collection of tuples, each containing host name, result (success of failure) + * @return Collection of start node results, each containing host name, result (success or failure) * and error message (if any). * @throws IgniteException In case of error. */ @IgniteAsyncSupported - public Collection> startNodes(File file, boolean restart, int timeout, + public Collection startNodes(File file, boolean restart, int timeout, int maxConn) throws IgniteException; /** @@ -272,12 +271,12 @@ public interface IgniteCluster extends ClusterGroup, IgniteAsyncSupport { * nodes on the host than expected. * @param timeout Connection timeout in milliseconds. * @param maxConn Number of parallel SSH connections to one host. - * @return Collection of tuples, each containing host name, result (success of failure) + * @return Collection of start node results, each containing host name, result (success or failure) * and error message (if any). * @throws IgniteException In case of error. */ @IgniteAsyncSupported - public Collection> startNodes(Collection> hosts, + public Collection startNodes(Collection> hosts, @Nullable Map dflts, boolean restart, int timeout, int maxConn) throws IgniteException; /** http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/7f4c5eed/modules/core/src/main/java/org/apache/ignite/cluster/ClusterStartNodeResult.java ---------------------------------------------------------------------- diff --git a/modules/core/src/main/java/org/apache/ignite/cluster/ClusterStartNodeResult.java b/modules/core/src/main/java/org/apache/ignite/cluster/ClusterStartNodeResult.java new file mode 100644 index 0000000..b6c3619 --- /dev/null +++ b/modules/core/src/main/java/org/apache/ignite/cluster/ClusterStartNodeResult.java @@ -0,0 +1,46 @@ +/* + * 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.cluster; + +import org.jetbrains.annotations.*; + +/** + * Cluster start node result information. + */ +public interface ClusterStartNodeResult { + /** + * Gets host name. + * + * @return Host name. + */ + public String getHostName(); + + /** + * Gets result of success or failure. + * + * @return Success or failure. True if success. + */ + public boolean isSuccess(); + + /** + * Gets error message if any. + * + * @return Error massage. + */ + @Nullable public String getError(); +} http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/7f4c5eed/modules/core/src/main/java/org/apache/ignite/internal/cluster/ClusterStartNodeResultImpl.java ---------------------------------------------------------------------- diff --git a/modules/core/src/main/java/org/apache/ignite/internal/cluster/ClusterStartNodeResultImpl.java b/modules/core/src/main/java/org/apache/ignite/internal/cluster/ClusterStartNodeResultImpl.java new file mode 100644 index 0000000..39301b3 --- /dev/null +++ b/modules/core/src/main/java/org/apache/ignite/internal/cluster/ClusterStartNodeResultImpl.java @@ -0,0 +1,87 @@ +/* + * 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.cluster; + +import org.apache.ignite.cluster.*; + +/** + * Implementation for cluster start node result. + */ +public class ClusterStartNodeResultImpl implements ClusterStartNodeResult { + /** Host name. */ + private String hostName; + + /** Result (success or failure). */ + private boolean success; + + /** Error message (if any) */ + private String error; + + /** + * @param hostName Host name. + * @param success Success or not. + * @param error Error message. + */ + public ClusterStartNodeResultImpl(String hostName, boolean success, String error) { + this.hostName = hostName; + this.success = success; + this.error = error; + } + + /** {@inheritDoc} */ + @Override public String getHostName() { + return hostName; + } + + /** + * Sets host name. + * + * @param hostName Host name. + */ + public void setHostName(String hostName) { + this.hostName = hostName; + } + + /** {@inheritDoc} */ + @Override public boolean isSuccess() { + return success; + } + + /** + * Sets success result. + * + * @param success Success result. + */ + public void setSuccess(boolean success) { + this.success = success; + } + + /** {@inheritDoc} */ + @Override public String getError() { + return error; + } + + /** + * Sets error message. + * + * @param error Error message. + */ + public void setError(String error) { + this.error = error; + } +} http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/7f4c5eed/modules/core/src/main/java/org/apache/ignite/internal/cluster/IgniteClusterAsyncImpl.java ---------------------------------------------------------------------- diff --git a/modules/core/src/main/java/org/apache/ignite/internal/cluster/IgniteClusterAsyncImpl.java b/modules/core/src/main/java/org/apache/ignite/internal/cluster/IgniteClusterAsyncImpl.java index 5a5daab..7f67b4f 100644 --- a/modules/core/src/main/java/org/apache/ignite/internal/cluster/IgniteClusterAsyncImpl.java +++ b/modules/core/src/main/java/org/apache/ignite/internal/cluster/IgniteClusterAsyncImpl.java @@ -20,7 +20,6 @@ package org.apache.ignite.internal.cluster; import org.apache.ignite.*; import org.apache.ignite.cluster.*; import org.apache.ignite.internal.*; -import org.apache.ignite.internal.util.lang.*; import org.apache.ignite.internal.util.typedef.internal.*; import org.apache.ignite.lang.*; import org.jetbrains.annotations.*; @@ -98,7 +97,7 @@ public class IgniteClusterAsyncImpl extends AsyncSupportAdapter } /** {@inheritDoc} */ - @Override public Collection> startNodes(File file, + @Override public Collection startNodes(File file, boolean restart, int timeout, int maxConn) @@ -112,7 +111,7 @@ public class IgniteClusterAsyncImpl extends AsyncSupportAdapter } /** {@inheritDoc} */ - @Override public Collection> startNodes( + @Override public Collection startNodes( Collection> hosts, @Nullable Map dflts, boolean restart, http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/7f4c5eed/modules/core/src/main/java/org/apache/ignite/internal/cluster/IgniteClusterImpl.java ---------------------------------------------------------------------- diff --git a/modules/core/src/main/java/org/apache/ignite/internal/cluster/IgniteClusterImpl.java b/modules/core/src/main/java/org/apache/ignite/internal/cluster/IgniteClusterImpl.java index 76a49dd..3c937b0 100644 --- a/modules/core/src/main/java/org/apache/ignite/internal/cluster/IgniteClusterImpl.java +++ b/modules/core/src/main/java/org/apache/ignite/internal/cluster/IgniteClusterImpl.java @@ -22,7 +22,6 @@ import org.apache.ignite.cluster.*; import org.apache.ignite.configuration.*; import org.apache.ignite.internal.*; import org.apache.ignite.internal.util.future.*; -import org.apache.ignite.internal.util.lang.*; import org.apache.ignite.internal.util.nodestart.*; import org.apache.ignite.internal.util.tostring.*; import org.apache.ignite.internal.util.typedef.*; @@ -189,7 +188,7 @@ public class IgniteClusterImpl extends ClusterGroupAdapter implements IgniteClus } /** {@inheritDoc} */ - @Override public Collection> startNodes(File file, + @Override public Collection startNodes(File file, boolean restart, int timeout, int maxConn) @@ -204,7 +203,7 @@ public class IgniteClusterImpl extends ClusterGroupAdapter implements IgniteClus } /** {@inheritDoc} */ - @Override public Collection> startNodes(Collection> hosts, + @Override public Collection startNodes(Collection> hosts, @Nullable Map dflts, boolean restart, int timeout, @@ -304,7 +303,7 @@ public class IgniteClusterImpl extends ClusterGroupAdapter implements IgniteClus * @return Future with results. * @see IgniteCluster#startNodes(java.io.File, boolean, int, int) */ - IgniteInternalFuture>> startNodesAsync(File file, + IgniteInternalFuture> startNodesAsync(File file, boolean restart, int timeout, int maxConn) @@ -332,7 +331,7 @@ public class IgniteClusterImpl extends ClusterGroupAdapter implements IgniteClus * @return Future with results. * @see IgniteCluster#startNodes(java.util.Collection, java.util.Map, boolean, int, int) */ - IgniteInternalFuture>> startNodesAsync( + IgniteInternalFuture> startNodesAsync( Collection> hosts, @Nullable Map dflts, boolean restart, @@ -348,7 +347,7 @@ public class IgniteClusterImpl extends ClusterGroupAdapter implements IgniteClus Map> specsMap = specifications(hosts, dflts); - Map> runMap = new HashMap<>(); + Map> runMap = new HashMap<>(); int nodeCallCnt = 0; @@ -393,7 +392,7 @@ public class IgniteClusterImpl extends ClusterGroupAdapter implements IgniteClus startIdx = neighbors.size() + 1; } - ConcurrentLinkedQueue nodeRuns = new ConcurrentLinkedQueue<>(); + ConcurrentLinkedQueue nodeRuns = new ConcurrentLinkedQueue<>(); runMap.put(host, nodeRuns); @@ -410,19 +409,17 @@ public class IgniteClusterImpl extends ClusterGroupAdapter implements IgniteClus // If there is nothing to start, return finished future with empty result. if (nodeCallCnt == 0) - return new GridFinishedFuture>>( - Collections.>emptyList()); + return new GridFinishedFuture>( + Collections.emptyList()); // Exceeding max line width for readability. - GridCompoundFuture, Collection>> - fut = new GridCompoundFuture<>( - CU.>objectsReducer() - ); + GridCompoundFuture> fut = + new GridCompoundFuture<>(CU.objectsReducer()); AtomicInteger cnt = new AtomicInteger(nodeCallCnt); // Limit maximum simultaneous connection number per host. - for (ConcurrentLinkedQueue queue : runMap.values()) { + for (ConcurrentLinkedQueue queue : runMap.values()) { for (int i = 0; i < maxConn; i++) { if (!runNextNodeCallable(queue, fut, cnt)) break; @@ -471,25 +468,25 @@ public class IgniteClusterImpl extends ClusterGroupAdapter implements IgniteClus * @param cnt Atomic counter to check if all futures are added to compound future. * @return {@code True} if task was started, {@code false} if queue was empty. */ - private boolean runNextNodeCallable(final ConcurrentLinkedQueue queue, - final GridCompoundFuture, Collection>> + private boolean runNextNodeCallable(final ConcurrentLinkedQueue queue, + final GridCompoundFuture> comp, final AtomicInteger cnt) { - IgniteNodeCallable call = queue.poll(); + StartNodeCallable call = queue.poll(); if (call == null) return false; - IgniteInternalFuture> fut = ctx.closure().callLocalSafe(call, true); + IgniteInternalFuture fut = ctx.closure().callLocalSafe(call, true); comp.add(fut); if (cnt.decrementAndGet() == 0) comp.markInitialized(); - fut.listen(new CI1>>() { - @Override public void apply(IgniteInternalFuture> f) { + fut.listen(new CI1>() { + @Override public void apply(IgniteInternalFuture f) { runNextNodeCallable(queue, comp, cnt); } }); http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/7f4c5eed/modules/core/src/main/java/org/apache/ignite/internal/util/nodestart/IgniteNodeCallable.java ---------------------------------------------------------------------- diff --git a/modules/core/src/main/java/org/apache/ignite/internal/util/nodestart/IgniteNodeCallable.java b/modules/core/src/main/java/org/apache/ignite/internal/util/nodestart/IgniteNodeCallable.java deleted file mode 100644 index 6c34efb..0000000 --- a/modules/core/src/main/java/org/apache/ignite/internal/util/nodestart/IgniteNodeCallable.java +++ /dev/null @@ -1,30 +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.util.nodestart; - -import org.apache.ignite.internal.util.lang.*; - -import java.util.concurrent.*; - -/** - * SSH-based node starter, returns tuple which contains hostname, success flag and error message - * if attempt was not successful. - */ -public interface IgniteNodeCallable extends Callable> { - // No-op. -} http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/7f4c5eed/modules/core/src/main/java/org/apache/ignite/internal/util/nodestart/IgniteSshHelper.java ---------------------------------------------------------------------- diff --git a/modules/core/src/main/java/org/apache/ignite/internal/util/nodestart/IgniteSshHelper.java b/modules/core/src/main/java/org/apache/ignite/internal/util/nodestart/IgniteSshHelper.java index 2cb620f..14a35b7 100644 --- a/modules/core/src/main/java/org/apache/ignite/internal/util/nodestart/IgniteSshHelper.java +++ b/modules/core/src/main/java/org/apache/ignite/internal/util/nodestart/IgniteSshHelper.java @@ -30,5 +30,5 @@ public interface IgniteSshHelper { * @param timeout Connection timeout. * @return {@link Callable} starting node using SSH. */ - public IgniteNodeCallable nodeStartCallable(IgniteRemoteStartSpecification spec, int timeout); + public StartNodeCallable nodeStartCallable(IgniteRemoteStartSpecification spec, int timeout); } http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/7f4c5eed/modules/core/src/main/java/org/apache/ignite/internal/util/nodestart/StartNodeCallable.java ---------------------------------------------------------------------- diff --git a/modules/core/src/main/java/org/apache/ignite/internal/util/nodestart/StartNodeCallable.java b/modules/core/src/main/java/org/apache/ignite/internal/util/nodestart/StartNodeCallable.java new file mode 100644 index 0000000..a1a2046 --- /dev/null +++ b/modules/core/src/main/java/org/apache/ignite/internal/util/nodestart/StartNodeCallable.java @@ -0,0 +1,30 @@ +/* + * 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.util.nodestart; + +import org.apache.ignite.cluster.*; + +import java.util.concurrent.*; + +/** + * SSH-based node starter, returns tuple which contains hostname, success flag and error message + * if attempt was not successful. + */ +public interface StartNodeCallable extends Callable { + // No-op. +} http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/7f4c5eed/modules/ssh/src/main/java/org/apache/ignite/internal/util/nodestart/IgniteNodeCallableImpl.java ---------------------------------------------------------------------- diff --git a/modules/ssh/src/main/java/org/apache/ignite/internal/util/nodestart/IgniteNodeCallableImpl.java b/modules/ssh/src/main/java/org/apache/ignite/internal/util/nodestart/IgniteNodeCallableImpl.java deleted file mode 100644 index 9b873bf..0000000 --- a/modules/ssh/src/main/java/org/apache/ignite/internal/util/nodestart/IgniteNodeCallableImpl.java +++ /dev/null @@ -1,308 +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.util.nodestart; - -import com.jcraft.jsch.*; -import org.apache.ignite.*; -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.resources.*; - -import java.io.*; -import java.text.*; -import java.util.*; - -import static org.apache.ignite.IgniteSystemProperties.*; - -/** - * SSH-based node starter. - */ -public class IgniteNodeCallableImpl implements IgniteNodeCallable { - /** Default Ignite home path for Windows (taken from environment variable). */ - private static final String DFLT_IGNITE_HOME_WIN = "%IGNITE_HOME%"; - - /** Default Ignite home path for Linux (taken from environment variable). */ - private static final String DFLT_IGNITE_HOME_LINUX = "$IGNITE_HOME"; - - /** Default start script path for Linux. */ - private static final String DFLT_SCRIPT_LINUX = "bin/ignite.sh -v"; - - /** Date format for log file name. */ - private static final SimpleDateFormat FILE_NAME_DATE_FORMAT = new SimpleDateFormat("MM-dd-yyyy--HH-mm-ss"); - - /** Specification. */ - private final IgniteRemoteStartSpecification spec; - - /** Connection timeout. */ - private final int timeout; - - /** Logger. */ - @LoggerResource - private IgniteLogger log; - - /** - * Required by Externalizable. - */ - public IgniteNodeCallableImpl() { - spec = null; - timeout = 0; - - assert false; - } - - /** - * Constructor. - * - * @param spec Specification. - * @param timeout Connection timeout. - */ - public IgniteNodeCallableImpl(IgniteRemoteStartSpecification spec, int timeout) { - assert spec != null; - - this.spec = spec; - this.timeout = timeout; - } - - /** {@inheritDoc} */ - @Override public GridTuple3 call() { - JSch ssh = new JSch(); - - Session ses = null; - - try { - if (spec.key() != null) - ssh.addIdentity(spec.key().getAbsolutePath()); - - ses = ssh.getSession(spec.username(), spec.host(), spec.port()); - - if (spec.password() != null) - ses.setPassword(spec.password()); - - ses.setConfig("StrictHostKeyChecking", "no"); - - ses.connect(timeout); - - boolean win = isWindows(ses); - - char separator = win ? '\\' : '/'; - - spec.fixPaths(separator); - - String igniteHome = spec.igniteHome(); - - if (igniteHome == null) - igniteHome = win ? DFLT_IGNITE_HOME_WIN : DFLT_IGNITE_HOME_LINUX; - - String script = spec.script(); - - if (script == null) - script = DFLT_SCRIPT_LINUX; - - String cfg = spec.configuration(); - - if (cfg == null) - cfg = ""; - - String startNodeCmd; - String scriptOutputFileName = FILE_NAME_DATE_FORMAT.format(new Date()) + '-' - + UUID.randomUUID().toString().substring(0, 8) + ".log"; - - if (win) - throw new UnsupportedOperationException("Apache Ignite cannot be auto-started on Windows from IgniteCluster.startNodes(…) API."); - else { // Assume Unix. - int spaceIdx = script.indexOf(' '); - - String scriptPath = spaceIdx > -1 ? script.substring(0, spaceIdx) : script; - String scriptArgs = spaceIdx > -1 ? script.substring(spaceIdx + 1) : ""; - String rmtLogArgs = buildRemoteLogArguments(spec.username(), spec.host()); - String tmpDir = env(ses, "$TMPDIR", "/tmp/"); - String scriptOutputDir = tmpDir + "ignite-startNodes"; - - shell(ses, "mkdir " + scriptOutputDir); - - // Mac os don't support ~ in double quotes. Trying get home path from remote system. - if (igniteHome.startsWith("~")) { - String homeDir = env(ses, "$HOME", "~"); - - igniteHome = igniteHome.replaceFirst("~", homeDir); - } - - startNodeCmd = new SB(). - // Console output is consumed, started nodes must use Ignite file appenders for log. - a("nohup "). - a("\"").a(igniteHome).a('/').a(scriptPath).a("\""). - a(" ").a(scriptArgs). - a(!cfg.isEmpty() ? " \"" : "").a(cfg).a(!cfg.isEmpty() ? "\"" : ""). - a(rmtLogArgs). - a(" > ").a(scriptOutputDir).a("/").a(scriptOutputFileName).a(" 2>& 1 &"). - toString(); - } - - info("Starting remote node with SSH command: " + startNodeCmd, spec.logger(), log); - - shell(ses, startNodeCmd); - - return new GridTuple3<>(spec.host(), true, null); - } - catch (IgniteInterruptedCheckedException e) { - return new GridTuple3<>(spec.host(), false, e.getMessage()); - } - catch (Exception e) { - return new GridTuple3<>(spec.host(), false, X.getFullStackTrace(e)); - } - finally { - if (ses != null && ses.isConnected()) - ses.disconnect(); - } - } - - /** - * Executes command using {@code shell} channel. - * - * @param ses SSH session. - * @param cmd Command. - * @throws JSchException In case of SSH error. - * @throws IOException If IO error occurs. - * @throws org.apache.ignite.internal.IgniteInterruptedCheckedException If thread was interrupted while waiting. - */ - private void shell(Session ses, String cmd) throws JSchException, IOException, IgniteInterruptedCheckedException { - ChannelShell ch = null; - - try { - ch = (ChannelShell)ses.openChannel("shell"); - - ch.connect(); - - try (PrintStream out = new PrintStream(ch.getOutputStream(), true)) { - out.println(cmd); - - U.sleep(1000); - } - } - finally { - if (ch != null && ch.isConnected()) - ch.disconnect(); - } - } - - /** - * Checks whether host is running Windows OS. - * - * @param ses SSH session. - * @return Whether host is running Windows OS. - * @throws JSchException In case of SSH error. - */ - private boolean isWindows(Session ses) throws JSchException { - try { - return exec(ses, "cmd.exe") != null; - } - catch (IOException ignored) { - return false; - } - } - - /** - * Gets the value of the specified environment variable. - * - * @param ses SSH session. - * @param name environment variable name. - * @param dflt default value. - * @return environment variable value. - * @throws JSchException In case of SSH error. - */ - private String env(Session ses, String name, String dflt) throws JSchException { - try { - return exec(ses, "echo " + name); - } - catch (IOException ignored) { - return dflt; - } - } - - /** - * Gets the value of the specified environment variable. - * - * @param ses SSH session. - * @param cmd environment variable name. - * @return environment variable value. - * @throws JSchException In case of SSH error. - * @throws IOException If failed. - */ - private String exec(Session ses, String cmd) throws JSchException, IOException { - ChannelExec ch = null; - - try { - ch = (ChannelExec)ses.openChannel("exec"); - - ch.setCommand(cmd); - - ch.connect(); - - try (BufferedReader reader = new BufferedReader(new InputStreamReader(ch.getInputStream()))) { - return reader.readLine(); - } - } - finally { - if (ch != null && ch.isConnected()) - ch.disconnect(); - } - } - - /** - * Builds ignite.sh attributes to set up SSH username and password and log directory for started node. - * - * @param username SSH user name. - * @param host Host. - * @return {@code ignite.sh} script arguments. - */ - private String buildRemoteLogArguments(String username, String host) { - assert username != null; - assert host != null; - - SB sb = new SB(); - - sb.a(" -J-D").a(IGNITE_SSH_HOST).a("=\"").a(host).a("\""). - a(" -J-D").a(IGNITE_SSH_USER_NAME).a("=\"").a(username).a("\""); - - return sb.toString(); - } - - /** - * @param log Logger. - * @return This callable for chaining method calls. - */ - public IgniteNodeCallable setLogger(IgniteLogger log) { - this.log = log; - - return this; - } - - /** - * Log info message to loggers. - * - * @param msg Message text. - * @param loggers Loggers. - */ - private void info(String msg, IgniteLogger... loggers) { - for (IgniteLogger logger : loggers) - if (logger != null && logger.isInfoEnabled()) - logger.info(msg); - } -} http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/7f4c5eed/modules/ssh/src/main/java/org/apache/ignite/internal/util/nodestart/IgniteSshHelperImpl.java ---------------------------------------------------------------------- diff --git a/modules/ssh/src/main/java/org/apache/ignite/internal/util/nodestart/IgniteSshHelperImpl.java b/modules/ssh/src/main/java/org/apache/ignite/internal/util/nodestart/IgniteSshHelperImpl.java index cccf678..c2370b6 100644 --- a/modules/ssh/src/main/java/org/apache/ignite/internal/util/nodestart/IgniteSshHelperImpl.java +++ b/modules/ssh/src/main/java/org/apache/ignite/internal/util/nodestart/IgniteSshHelperImpl.java @@ -22,7 +22,7 @@ package org.apache.ignite.internal.util.nodestart; */ public class IgniteSshHelperImpl implements IgniteSshHelper { /** {@inheritDoc} */ - @Override public IgniteNodeCallable nodeStartCallable(IgniteRemoteStartSpecification spec, int timeout) { - return new IgniteNodeCallableImpl(spec, timeout); + @Override public StartNodeCallable nodeStartCallable(IgniteRemoteStartSpecification spec, int timeout) { + return new StartNodeCallableImpl(spec, timeout); } } http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/7f4c5eed/modules/ssh/src/main/java/org/apache/ignite/internal/util/nodestart/StartNodeCallableImpl.java ---------------------------------------------------------------------- diff --git a/modules/ssh/src/main/java/org/apache/ignite/internal/util/nodestart/StartNodeCallableImpl.java b/modules/ssh/src/main/java/org/apache/ignite/internal/util/nodestart/StartNodeCallableImpl.java new file mode 100644 index 0000000..7c6010f --- /dev/null +++ b/modules/ssh/src/main/java/org/apache/ignite/internal/util/nodestart/StartNodeCallableImpl.java @@ -0,0 +1,309 @@ +/* + * 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.util.nodestart; + +import com.jcraft.jsch.*; +import org.apache.ignite.*; +import org.apache.ignite.cluster.*; +import org.apache.ignite.internal.*; +import org.apache.ignite.internal.cluster.*; +import org.apache.ignite.internal.util.typedef.*; +import org.apache.ignite.internal.util.typedef.internal.*; +import org.apache.ignite.resources.*; + +import java.io.*; +import java.text.*; +import java.util.*; + +import static org.apache.ignite.IgniteSystemProperties.*; + +/** + * SSH-based node starter. + */ +public class StartNodeCallableImpl implements StartNodeCallable { + /** Default Ignite home path for Windows (taken from environment variable). */ + private static final String DFLT_IGNITE_HOME_WIN = "%IGNITE_HOME%"; + + /** Default Ignite home path for Linux (taken from environment variable). */ + private static final String DFLT_IGNITE_HOME_LINUX = "$IGNITE_HOME"; + + /** Default start script path for Linux. */ + private static final String DFLT_SCRIPT_LINUX = "bin/ignite.sh -v"; + + /** Date format for log file name. */ + private static final SimpleDateFormat FILE_NAME_DATE_FORMAT = new SimpleDateFormat("MM-dd-yyyy--HH-mm-ss"); + + /** Specification. */ + private final IgniteRemoteStartSpecification spec; + + /** Connection timeout. */ + private final int timeout; + + /** Logger. */ + @LoggerResource + private IgniteLogger log; + + /** + * Required by Externalizable. + */ + public StartNodeCallableImpl() { + spec = null; + timeout = 0; + + assert false; + } + + /** + * Constructor. + * + * @param spec Specification. + * @param timeout Connection timeout. + */ + public StartNodeCallableImpl(IgniteRemoteStartSpecification spec, int timeout) { + assert spec != null; + + this.spec = spec; + this.timeout = timeout; + } + + /** {@inheritDoc} */ + @Override public ClusterStartNodeResult call() { + JSch ssh = new JSch(); + + Session ses = null; + + try { + if (spec.key() != null) + ssh.addIdentity(spec.key().getAbsolutePath()); + + ses = ssh.getSession(spec.username(), spec.host(), spec.port()); + + if (spec.password() != null) + ses.setPassword(spec.password()); + + ses.setConfig("StrictHostKeyChecking", "no"); + + ses.connect(timeout); + + boolean win = isWindows(ses); + + char separator = win ? '\\' : '/'; + + spec.fixPaths(separator); + + String igniteHome = spec.igniteHome(); + + if (igniteHome == null) + igniteHome = win ? DFLT_IGNITE_HOME_WIN : DFLT_IGNITE_HOME_LINUX; + + String script = spec.script(); + + if (script == null) + script = DFLT_SCRIPT_LINUX; + + String cfg = spec.configuration(); + + if (cfg == null) + cfg = ""; + + String startNodeCmd; + String scriptOutputFileName = FILE_NAME_DATE_FORMAT.format(new Date()) + '-' + + UUID.randomUUID().toString().substring(0, 8) + ".log"; + + if (win) + throw new UnsupportedOperationException("Apache Ignite cannot be auto-started on Windows from IgniteCluster.startNodes(…) API."); + else { // Assume Unix. + int spaceIdx = script.indexOf(' '); + + String scriptPath = spaceIdx > -1 ? script.substring(0, spaceIdx) : script; + String scriptArgs = spaceIdx > -1 ? script.substring(spaceIdx + 1) : ""; + String rmtLogArgs = buildRemoteLogArguments(spec.username(), spec.host()); + String tmpDir = env(ses, "$TMPDIR", "/tmp/"); + String scriptOutputDir = tmpDir + "ignite-startNodes"; + + shell(ses, "mkdir " + scriptOutputDir); + + // Mac os don't support ~ in double quotes. Trying get home path from remote system. + if (igniteHome.startsWith("~")) { + String homeDir = env(ses, "$HOME", "~"); + + igniteHome = igniteHome.replaceFirst("~", homeDir); + } + + startNodeCmd = new SB(). + // Console output is consumed, started nodes must use Ignite file appenders for log. + a("nohup "). + a("\"").a(igniteHome).a('/').a(scriptPath).a("\""). + a(" ").a(scriptArgs). + a(!cfg.isEmpty() ? " \"" : "").a(cfg).a(!cfg.isEmpty() ? "\"" : ""). + a(rmtLogArgs). + a(" > ").a(scriptOutputDir).a("/").a(scriptOutputFileName).a(" 2>& 1 &"). + toString(); + } + + info("Starting remote node with SSH command: " + startNodeCmd, spec.logger(), log); + + shell(ses, startNodeCmd); + + return new ClusterStartNodeResultImpl(spec.host(), true, null); + } + catch (IgniteInterruptedCheckedException e) { + return new ClusterStartNodeResultImpl(spec.host(), false, e.getMessage()); + } + catch (Exception e) { + return new ClusterStartNodeResultImpl(spec.host(), false, X.getFullStackTrace(e)); + } + finally { + if (ses != null && ses.isConnected()) + ses.disconnect(); + } + } + + /** + * Executes command using {@code shell} channel. + * + * @param ses SSH session. + * @param cmd Command. + * @throws JSchException In case of SSH error. + * @throws IOException If IO error occurs. + * @throws IgniteInterruptedCheckedException If thread was interrupted while waiting. + */ + private void shell(Session ses, String cmd) throws JSchException, IOException, IgniteInterruptedCheckedException { + ChannelShell ch = null; + + try { + ch = (ChannelShell)ses.openChannel("shell"); + + ch.connect(); + + try (PrintStream out = new PrintStream(ch.getOutputStream(), true)) { + out.println(cmd); + + U.sleep(1000); + } + } + finally { + if (ch != null && ch.isConnected()) + ch.disconnect(); + } + } + + /** + * Checks whether host is running Windows OS. + * + * @param ses SSH session. + * @return Whether host is running Windows OS. + * @throws JSchException In case of SSH error. + */ + private boolean isWindows(Session ses) throws JSchException { + try { + return exec(ses, "cmd.exe") != null; + } + catch (IOException ignored) { + return false; + } + } + + /** + * Gets the value of the specified environment variable. + * + * @param ses SSH session. + * @param name environment variable name. + * @param dflt default value. + * @return environment variable value. + * @throws JSchException In case of SSH error. + */ + private String env(Session ses, String name, String dflt) throws JSchException { + try { + return exec(ses, "echo " + name); + } + catch (IOException ignored) { + return dflt; + } + } + + /** + * Gets the value of the specified environment variable. + * + * @param ses SSH session. + * @param cmd environment variable name. + * @return environment variable value. + * @throws JSchException In case of SSH error. + * @throws IOException If failed. + */ + private String exec(Session ses, String cmd) throws JSchException, IOException { + ChannelExec ch = null; + + try { + ch = (ChannelExec)ses.openChannel("exec"); + + ch.setCommand(cmd); + + ch.connect(); + + try (BufferedReader reader = new BufferedReader(new InputStreamReader(ch.getInputStream()))) { + return reader.readLine(); + } + } + finally { + if (ch != null && ch.isConnected()) + ch.disconnect(); + } + } + + /** + * Builds ignite.sh attributes to set up SSH username and password and log directory for started node. + * + * @param username SSH user name. + * @param host Host. + * @return {@code ignite.sh} script arguments. + */ + private String buildRemoteLogArguments(String username, String host) { + assert username != null; + assert host != null; + + SB sb = new SB(); + + sb.a(" -J-D").a(IGNITE_SSH_HOST).a("=\"").a(host).a("\""). + a(" -J-D").a(IGNITE_SSH_USER_NAME).a("=\"").a(username).a("\""); + + return sb.toString(); + } + + /** + * @param log Logger. + * @return This callable for chaining method calls. + */ + public StartNodeCallable setLogger(IgniteLogger log) { + this.log = log; + + return this; + } + + /** + * Log info message to loggers. + * + * @param msg Message text. + * @param loggers Loggers. + */ + private void info(String msg, IgniteLogger... loggers) { + for (IgniteLogger logger : loggers) + if (logger != null && logger.isInfoEnabled()) + logger.info(msg); + } +} http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/7f4c5eed/modules/ssh/src/test/java/org/apache/ignite/internal/IgniteProjectionStartStopRestartSelfTest.java ---------------------------------------------------------------------- diff --git a/modules/ssh/src/test/java/org/apache/ignite/internal/IgniteProjectionStartStopRestartSelfTest.java b/modules/ssh/src/test/java/org/apache/ignite/internal/IgniteProjectionStartStopRestartSelfTest.java index d4c5fc5..7fa5b4b 100644 --- a/modules/ssh/src/test/java/org/apache/ignite/internal/IgniteProjectionStartStopRestartSelfTest.java +++ b/modules/ssh/src/test/java/org/apache/ignite/internal/IgniteProjectionStartStopRestartSelfTest.java @@ -20,7 +20,6 @@ package org.apache.ignite.internal; import org.apache.ignite.*; import org.apache.ignite.cluster.*; import org.apache.ignite.events.*; -import org.apache.ignite.internal.util.lang.*; import org.apache.ignite.internal.util.nodestart.*; import org.apache.ignite.internal.util.typedef.*; import org.apache.ignite.internal.util.typedef.internal.*; @@ -178,19 +177,19 @@ public class IgniteProjectionStartStopRestartSelfTest extends GridCommonAbstract public void testStartOneNode() throws Exception { joinedLatch = new CountDownLatch(1); - Collection> res = + Collection res = startNodes(ignite.cluster(), maps(Collections.singleton(HOST), SSH_UNAME, pwd, key, 1, U.getIgniteHome(), CFG_NO_ATTR, null), null, false, 0, 16); assert res.size() == 1; - F.forEach(res, new CI1>() { - @Override public void apply(GridTuple3 t) { - assert t.get1().equals(HOST); + F.forEach(res, new CI1() { + @Override public void apply(ClusterStartNodeResult t) { + assert t.getHostName().equals(HOST); - if (!t.get2()) - throw new IgniteException(t.get3()); + if (!t.isSuccess()) + throw new IgniteException(t.getError()); } }); @@ -208,19 +207,19 @@ public class IgniteProjectionStartStopRestartSelfTest extends GridCommonAbstract public void testStartThreeNodes() throws Exception { joinedLatch = new CountDownLatch(3); - Collection> res = + Collection res = startNodes(ignite.cluster(), maps(Collections.singleton(HOST), SSH_UNAME, pwd, key, 3, U.getIgniteHome(), CFG_NO_ATTR, null), null, false, DFLT_TIMEOUT, 1); assert res.size() == 3; - F.forEach(res, new CI1>() { - @Override public void apply(GridTuple3 t) { - assert t.get1().equals(HOST); + F.forEach(res, new CI1() { + @Override public void apply(ClusterStartNodeResult t) { + assert t.getHostName().equals(HOST); - if (!t.get2()) - throw new IgniteException(t.get3()); + if (!t.isSuccess()) + throw new IgniteException(t.getError()); } }); @@ -238,19 +237,19 @@ public class IgniteProjectionStartStopRestartSelfTest extends GridCommonAbstract public void testStartThreeNodesAndDoEmptyCall() throws Exception { joinedLatch = new CountDownLatch(3); - Collection> res = + Collection res = startNodes(ignite.cluster(), maps(Collections.singleton(HOST), SSH_UNAME, pwd, key, 3, U.getIgniteHome(), CFG_NO_ATTR, null), null, false, 0, 16); assert res.size() == 3; - F.forEach(res, new CI1>() { - @Override public void apply(GridTuple3 t) { - assert t.get1().equals(HOST); + F.forEach(res, new CI1() { + @Override public void apply(ClusterStartNodeResult t) { + assert t.getHostName().equals(HOST); - if (!t.get2()) - throw new IgniteException(t.get3()); + if (!t.isSuccess()) + throw new IgniteException(t.getError()); } }); @@ -279,19 +278,19 @@ public class IgniteProjectionStartStopRestartSelfTest extends GridCommonAbstract public void testStartThreeNodesAndTryToStartOneNode() throws Exception { joinedLatch = new CountDownLatch(3); - Collection> res = + Collection res = startNodes(ignite.cluster(), maps(Collections.singleton(HOST), SSH_UNAME, pwd, key, 3, U.getIgniteHome(), CFG_NO_ATTR, null), null, false, 0, 16); assert res.size() == 3; - F.forEach(res, new CI1>() { - @Override public void apply(GridTuple3 t) { - assert t.get1().equals(HOST); + F.forEach(res, new CI1() { + @Override public void apply(ClusterStartNodeResult t) { + assert t.getHostName().equals(HOST); - if (!t.get2()) - throw new IgniteException(t.get3()); + if (!t.isSuccess()) + throw new IgniteException(t.getError()); } }); @@ -320,19 +319,19 @@ public class IgniteProjectionStartStopRestartSelfTest extends GridCommonAbstract public void testStartFiveNodesInTwoCalls() throws Exception { joinedLatch = new CountDownLatch(3); - Collection> res = + Collection res = startNodes(ignite.cluster(), maps(Collections.singleton(HOST), SSH_UNAME, pwd, key, 3, U.getIgniteHome(), CFG_NO_ATTR, null), null, false, 0, 16); assert res.size() == 3; - F.forEach(res, new CI1>() { - @Override public void apply(GridTuple3 t) { - assert t.get1().equals(HOST); + F.forEach(res, new CI1() { + @Override public void apply(ClusterStartNodeResult t) { + assert t.getHostName().equals(HOST); - if (!t.get2()) - throw new IgniteException(t.get3()); + if (!t.isSuccess()) + throw new IgniteException(t.getError()); } }); @@ -351,12 +350,12 @@ public class IgniteProjectionStartStopRestartSelfTest extends GridCommonAbstract assert res.size() == 2; - F.forEach(res, new CI1>() { - @Override public void apply(GridTuple3 t) { - assert t.get1().equals(HOST); + F.forEach(res, new CI1() { + @Override public void apply(ClusterStartNodeResult t) { + assert t.getHostName().equals(HOST); - if (!t.get2()) - throw new IgniteException(t.get3()); + if (!t.isSuccess()) + throw new IgniteException(t.getError()); } }); @@ -374,7 +373,7 @@ public class IgniteProjectionStartStopRestartSelfTest extends GridCommonAbstract public void testStartFiveWithTwoSpecs() throws Exception { joinedLatch = new CountDownLatch(5); - Collection> res = + Collection res = startNodes(ignite.cluster(), F.asList(map(HOST, SSH_UNAME, pwd, key, 2, U.getIgniteHome(), CFG_NO_ATTR, null), map(HOST, SSH_UNAME, pwd, key, 3, U.getIgniteHome(), CFG_NO_ATTR, null)), @@ -382,12 +381,12 @@ public class IgniteProjectionStartStopRestartSelfTest extends GridCommonAbstract assert res.size() == 5; - F.forEach(res, new CI1>() { - @Override public void apply(GridTuple3 t) { - assert t.get1().equals(HOST); + F.forEach(res, new CI1() { + @Override public void apply(ClusterStartNodeResult t) { + assert t.getHostName().equals(HOST); - if (!t.get2()) - throw new IgniteException(t.get3()); + if (!t.isSuccess()) + throw new IgniteException(t.getError()); } }); @@ -405,19 +404,19 @@ public class IgniteProjectionStartStopRestartSelfTest extends GridCommonAbstract public void testStartThreeNodesAndRestart() throws Exception { joinedLatch = new CountDownLatch(3); - Collection> res = + Collection res = startNodes(ignite.cluster(), maps(Collections.singleton(HOST), SSH_UNAME, pwd, key, 3, U.getIgniteHome(), CFG_NO_ATTR, null), null, false, 0, 16); assert res.size() == 3; - F.forEach(res, new CI1>() { - @Override public void apply(GridTuple3 t) { - assert t.get1().equals(HOST); + F.forEach(res, new CI1() { + @Override public void apply(ClusterStartNodeResult t) { + assert t.getHostName().equals(HOST); - if (!t.get2()) - throw new IgniteException(t.get3()); + if (!t.isSuccess()) + throw new IgniteException(t.getError()); } }); @@ -437,12 +436,12 @@ public class IgniteProjectionStartStopRestartSelfTest extends GridCommonAbstract assert res.size() == 3; - F.forEach(res, new CI1>() { - @Override public void apply(GridTuple3 t) { - assert t.get1().equals(HOST); + F.forEach(res, new CI1() { + @Override public void apply(ClusterStartNodeResult t) { + assert t.getHostName().equals(HOST); - if (!t.get2()) - throw new IgniteException(t.get3()); + if (!t.isSuccess()) + throw new IgniteException(t.getError()); } }); @@ -465,19 +464,19 @@ public class IgniteProjectionStartStopRestartSelfTest extends GridCommonAbstract script = Paths.get(U.getIgniteHome()).relativize(U.resolveIgnitePath(script).toPath()).toString(); - Collection> res = + Collection res = startNodes(ignite.cluster(), maps(Collections.singleton(HOST), SSH_UNAME, pwd, key, 1, U.getIgniteHome(), null, script), null, false, 0, 16); assert res.size() == 1; - F.forEach(res, new CI1>() { - @Override public void apply(GridTuple3 t) { - assert t.get1().equals(HOST); + F.forEach(res, new CI1() { + @Override public void apply(ClusterStartNodeResult t) { + assert t.getHostName().equals(HOST); - if (!t.get2()) - throw new IgniteException(t.get3()); + if (!t.isSuccess()) + throw new IgniteException(t.getError()); } }); @@ -497,19 +496,19 @@ public class IgniteProjectionStartStopRestartSelfTest extends GridCommonAbstract public void testStopNodes() throws Exception { joinedLatch = new CountDownLatch(3); - Collection> res = + Collection res = startNodes(ignite.cluster(), maps(Collections.singleton(HOST), SSH_UNAME, pwd, null, 3, U.getIgniteHome(), CFG_NO_ATTR, null), null, false, 0, 16); assert res.size() == 3; - F.forEach(res, new CI1>() { - @Override public void apply(GridTuple3 t) { - assert t.get1().equals(HOST); + F.forEach(res, new CI1() { + @Override public void apply(ClusterStartNodeResult t) { + assert t.getHostName().equals(HOST); - if (!t.get2()) - throw new IgniteException(t.get3()); + if (!t.isSuccess()) + throw new IgniteException(t.getError()); } }); @@ -532,19 +531,19 @@ public class IgniteProjectionStartStopRestartSelfTest extends GridCommonAbstract public void testStopNodesFiltered() throws Exception { joinedLatch = new CountDownLatch(2); - Collection> res = + Collection res = startNodes(ignite.cluster(), maps(Collections.singleton(HOST), SSH_UNAME, pwd, key, 2, U.getIgniteHome(), CFG_ATTR, null), null, false, 0, 16); assert res.size() == 2; - F.forEach(res, new CI1>() { - @Override public void apply(GridTuple3 t) { - assert t.get1().equals(HOST); + F.forEach(res, new CI1() { + @Override public void apply(ClusterStartNodeResult t) { + assert t.getHostName().equals(HOST); - if (!t.get2()) - throw new IgniteException(t.get3()); + if (!t.isSuccess()) + throw new IgniteException(t.getError()); } }); @@ -558,12 +557,12 @@ public class IgniteProjectionStartStopRestartSelfTest extends GridCommonAbstract assert res.size() == 1; - F.forEach(res, new CI1>() { - @Override public void apply(GridTuple3 t) { - assert t.get1().equals(HOST); + F.forEach(res, new CI1() { + @Override public void apply(ClusterStartNodeResult t) { + assert t.getHostName().equals(HOST); - if (!t.get2()) - throw new IgniteException(t.get3()); + if (!t.isSuccess()) + throw new IgniteException(t.getError()); } }); @@ -595,19 +594,19 @@ public class IgniteProjectionStartStopRestartSelfTest extends GridCommonAbstract public void testStopNodeById() throws Exception { joinedLatch = new CountDownLatch(3); - Collection> res = + Collection res = startNodes(ignite.cluster(), maps(Collections.singleton(HOST), SSH_UNAME, pwd, key, 3, U.getIgniteHome(), CFG_NO_ATTR, null), null, false, 0, 16); assert res.size() == 3; - F.forEach(res, new CI1>() { - @Override public void apply(GridTuple3 t) { - assert t.get1().equals(HOST); + F.forEach(res, new CI1() { + @Override public void apply(ClusterStartNodeResult t) { + assert t.getHostName().equals(HOST); - if (!t.get2()) - throw new IgniteException(t.get3()); + if (!t.isSuccess()) + throw new IgniteException(t.getError()); } }); @@ -630,19 +629,19 @@ public class IgniteProjectionStartStopRestartSelfTest extends GridCommonAbstract public void testStopNodesByIds() throws Exception { joinedLatch = new CountDownLatch(3); - Collection> res = + Collection res = startNodes(ignite.cluster(), maps(Collections.singleton(HOST), SSH_UNAME, pwd, key, 3, U.getIgniteHome(), CFG_NO_ATTR, null), null, false, 0, 16); assert res.size() == 3; - F.forEach(res, new CI1>() { - @Override public void apply(GridTuple3 t) { - assert t.get1().equals(HOST); + F.forEach(res, new CI1() { + @Override public void apply(ClusterStartNodeResult t) { + assert t.getHostName().equals(HOST); - if (!t.get2()) - throw new IgniteException(t.get3()); + if (!t.isSuccess()) + throw new IgniteException(t.getError()); } }); @@ -672,19 +671,19 @@ public class IgniteProjectionStartStopRestartSelfTest extends GridCommonAbstract public void testStopNodesByIdsC() throws Exception { joinedLatch = new CountDownLatch(3); - Collection> res = + Collection res = startNodes(ignite.cluster(), maps(Collections.singleton(HOST), SSH_UNAME, pwd, key, 3, U.getIgniteHome(), CFG_NO_ATTR, null), null, false, 0, 16); assert res.size() == 3; - F.forEach(res, new CI1>() { - @Override public void apply(GridTuple3 t) { - assert t.get1().equals(HOST); + F.forEach(res, new CI1() { + @Override public void apply(ClusterStartNodeResult t) { + assert t.getHostName().equals(HOST); - if (!t.get2()) - throw new IgniteException(t.get3()); + if (!t.isSuccess()) + throw new IgniteException(t.getError()); } }); @@ -709,19 +708,19 @@ public class IgniteProjectionStartStopRestartSelfTest extends GridCommonAbstract public void testRestartNodes() throws Exception { joinedLatch = new CountDownLatch(3); - Collection> res = + Collection res = startNodes(ignite.cluster(), maps(Collections.singleton(HOST), SSH_UNAME, pwd, key, 3, U.getIgniteHome(), CFG_NO_ATTR, null), null, false, 0, 16); assert res.size() == 3; - F.forEach(res, new CI1>() { - @Override public void apply(GridTuple3 t) { - assert t.get1().equals(HOST); + F.forEach(res, new CI1() { + @Override public void apply(ClusterStartNodeResult t) { + assert t.getHostName().equals(HOST); - if (!t.get2()) - throw new IgniteException(t.get3()); + if (!t.isSuccess()) + throw new IgniteException(t.getError()); } }); @@ -746,19 +745,19 @@ public class IgniteProjectionStartStopRestartSelfTest extends GridCommonAbstract public void testRestartNodesFiltered() throws Exception { joinedLatch = new CountDownLatch(2); - Collection> res = + Collection res = startNodes(ignite.cluster(), maps(Collections.singleton(HOST), SSH_UNAME, pwd, key, 2, U.getIgniteHome(), CFG_ATTR, null), null, false, 0, 16); assert res.size() == 2; - F.forEach(res, new CI1>() { - @Override public void apply(GridTuple3 t) { - assert t.get1().equals(HOST); + F.forEach(res, new CI1() { + @Override public void apply(ClusterStartNodeResult t) { + assert t.getHostName().equals(HOST); - if (!t.get2()) - throw new IgniteException(t.get3()); + if (!t.isSuccess()) + throw new IgniteException(t.getError()); } }); @@ -772,12 +771,12 @@ public class IgniteProjectionStartStopRestartSelfTest extends GridCommonAbstract assert res.size() == 1; - F.forEach(res, new CI1>() { - @Override public void apply(GridTuple3 t) { - assert t.get1().equals(HOST); + F.forEach(res, new CI1() { + @Override public void apply(ClusterStartNodeResult t) { + assert t.getHostName().equals(HOST); - if (!t.get2()) - throw new IgniteException(t.get3()); + if (!t.isSuccess()) + throw new IgniteException(t.getError()); } }); @@ -812,19 +811,19 @@ public class IgniteProjectionStartStopRestartSelfTest extends GridCommonAbstract public void testRestartNodeById() throws Exception { joinedLatch = new CountDownLatch(3); - Collection> res = + Collection res = startNodes(ignite.cluster(), maps(Collections.singleton(HOST), SSH_UNAME, pwd, key, 3, U.getIgniteHome(), CFG_NO_ATTR, null), null, false, 0, 16); assert res.size() == 3; - F.forEach(res, new CI1>() { - @Override public void apply(GridTuple3 t) { - assert t.get1().equals(HOST); + F.forEach(res, new CI1() { + @Override public void apply(ClusterStartNodeResult t) { + assert t.getHostName().equals(HOST); - if (!t.get2()) - throw new IgniteException(t.get3()); + if (!t.isSuccess()) + throw new IgniteException(t.getError()); } }); @@ -849,19 +848,19 @@ public class IgniteProjectionStartStopRestartSelfTest extends GridCommonAbstract public void testRestartNodesByIds() throws Exception { joinedLatch = new CountDownLatch(3); - Collection> res = + Collection res = startNodes(ignite.cluster(), maps(Collections.singleton(HOST), SSH_UNAME, pwd, key, 3, U.getIgniteHome(), CFG_NO_ATTR, null), null, false, 0, 16); assert res.size() == 3; - F.forEach(res, new CI1>() { - @Override public void apply(GridTuple3 t) { - assert t.get1().equals(HOST); + F.forEach(res, new CI1() { + @Override public void apply(ClusterStartNodeResult t) { + assert t.getHostName().equals(HOST); - if (!t.get2()) - throw new IgniteException(t.get3()); + if (!t.isSuccess()) + throw new IgniteException(t.getError()); } }); @@ -888,19 +887,19 @@ public class IgniteProjectionStartStopRestartSelfTest extends GridCommonAbstract public void testRestartNodesByIdsC() throws Exception { joinedLatch = new CountDownLatch(3); - Collection> res = + Collection res = startNodes(ignite.cluster(), maps(Collections.singleton(HOST), SSH_UNAME, pwd, key, 3, U.getIgniteHome(), CFG_NO_ATTR, null), null, false, 0, 16); assert res.size() == 3; - F.forEach(res, new CI1>() { - @Override public void apply(GridTuple3 t) { - assert t.get1().equals(HOST); + F.forEach(res, new CI1() { + @Override public void apply(ClusterStartNodeResult t) { + assert t.getHostName().equals(HOST); - if (!t.get2()) - throw new IgniteException(t.get3()); + if (!t.isSuccess()) + throw new IgniteException(t.getError()); } }); @@ -1016,7 +1015,7 @@ public class IgniteProjectionStartStopRestartSelfTest extends GridCommonAbstract * @param maxConn Maximum connections. * @return Results collection. */ - private Collection> startNodes(IgniteCluster cluster, + private Collection startNodes(IgniteCluster cluster, Collection> hosts, @Nullable Map dflts, boolean restart, @@ -1026,6 +1025,6 @@ public class IgniteProjectionStartStopRestartSelfTest extends GridCommonAbstract assertNull(cluster.startNodes(hosts, dflts, restart, timeout, maxConn)); - return cluster.>>future().get(WAIT_TIMEOUT); + return cluster.>future().get(WAIT_TIMEOUT); } } http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/7f4c5eed/modules/ssh/src/test/java/org/apache/ignite/internal/IgniteStartStopRestartTestSuite.java ---------------------------------------------------------------------- diff --git a/modules/ssh/src/test/java/org/apache/ignite/internal/IgniteStartStopRestartTestSuite.java b/modules/ssh/src/test/java/org/apache/ignite/internal/IgniteStartStopRestartTestSuite.java index 65fcb44..6d09eb2 100644 --- a/modules/ssh/src/test/java/org/apache/ignite/internal/IgniteStartStopRestartTestSuite.java +++ b/modules/ssh/src/test/java/org/apache/ignite/internal/IgniteStartStopRestartTestSuite.java @@ -37,3 +37,4 @@ public class IgniteStartStopRestartTestSuite { return suite; } } + http://git-wip-us.apache.org/repos/asf/incubator-ignite/blob/7f4c5eed/modules/visor-console/src/main/scala/org/apache/ignite/visor/commands/start/VisorStartCommand.scala ---------------------------------------------------------------------- diff --git a/modules/visor-console/src/main/scala/org/apache/ignite/visor/commands/start/VisorStartCommand.scala b/modules/visor-console/src/main/scala/org/apache/ignite/visor/commands/start/VisorStartCommand.scala index 5342fc1..a6de108 100644 --- a/modules/visor-console/src/main/scala/org/apache/ignite/visor/commands/start/VisorStartCommand.scala +++ b/modules/visor-console/src/main/scala/org/apache/ignite/visor/commands/start/VisorStartCommand.scala @@ -228,7 +228,7 @@ class VisorStartCommand { try res = ignite.cluster.startNodes(file, restart, timeout, maxConn).map(t => { - Result(t.get1, t.get2, t.get3) + Result(t.getHostName, t.isSuccess, t.getError) }).toSeq catch { case e: IgniteException => scold(e.getMessage).^^ @@ -284,7 +284,7 @@ class VisorStartCommand { try res = ignite.cluster.startNodes(asJavaCollection(Seq(params)), null, restart, timeout, maxConn). - map(t => Result(t.get1, t.get2, t.get3)).toSeq + map(t => Result(t.getHostName, t.isSuccess, t.getError)).toSeq catch { case e: IgniteException => scold(e.getMessage).^^ case _: RejectedExecutionException => scold("Failed due to system error.").^^