added runScriptOnNodesWithTag methods. theoretically it is supported for any cloud as a part of compute service. (issue 222)

This commit is contained in:
Alex Yarmula 2010-04-10 19:11:01 -07:00
parent d4d9d76388
commit 882bf5f651
7 changed files with 858 additions and 622 deletions

View File

@ -22,11 +22,13 @@ import static org.testng.Assert.assertEquals;
import org.jclouds.compute.BaseComputeServiceLiveTest; import org.jclouds.compute.BaseComputeServiceLiveTest;
import org.jclouds.compute.domain.*; import org.jclouds.compute.domain.*;
import org.jclouds.domain.Credentials;
import org.jclouds.ssh.jsch.config.JschSshClientModule; import org.jclouds.ssh.jsch.config.JschSshClientModule;
import org.testng.annotations.BeforeClass; import org.testng.annotations.BeforeClass;
import org.testng.annotations.Test; import org.testng.annotations.Test;
import com.google.common.collect.Iterables; import com.google.common.collect.Iterables;
import java.util.Map; import java.util.Map;
/** /**
@ -62,11 +64,15 @@ public class EC2ComputeServiceLiveTest extends BaseComputeServiceLiveTest {
} }
@Test @Test
public void testCredentialsMapping() { public void testScriptExecution() throws Exception {
Template simpleTemplate = client.templateBuilder().smallest().build(); Template simpleTemplate = client.templateBuilder().smallest().build();
// client.runNodesWithTag("ec2", 1, simpleTemplate); client.runNodesWithTag("ec2", 1, simpleTemplate);
Map<String, ? extends NodeMetadata> map = client.getNodesWithTag("ec2"); Map<String, ? extends NodeMetadata> map = client.getNodesWithTag("ec2");
int a = 5; NodeMetadata node = map.values().iterator().next();
Credentials creds = new Credentials("ubuntu", keyPair.get("public"));
client.runScriptOnNodesWithTag("ec2", creds,
"mkdir ~/ahha; sleep 3".getBytes());
client.destroyNodesWithTag("ec2");
} }
@Override @Override

View File

@ -27,9 +27,12 @@ import org.jclouds.compute.domain.Size;
import org.jclouds.compute.domain.Template; import org.jclouds.compute.domain.Template;
import org.jclouds.compute.domain.TemplateBuilder; import org.jclouds.compute.domain.TemplateBuilder;
import org.jclouds.compute.internal.BaseComputeService; import org.jclouds.compute.internal.BaseComputeService;
import org.jclouds.compute.options.RunScriptOptions;
import org.jclouds.domain.Credentials;
import org.jclouds.domain.Location; import org.jclouds.domain.Location;
import com.google.inject.ImplementedBy; import com.google.inject.ImplementedBy;
import org.jclouds.ssh.ExecResponse;
/** /**
* Provides portable access to launching compute instances. * Provides portable access to launching compute instances.
@ -152,4 +155,26 @@ public interface ComputeService {
*/ */
Map<String, ? extends NodeMetadata> getNodesWithTag(String tag); Map<String, ? extends NodeMetadata> getNodesWithTag(String tag);
/**
* Runs the script without any additional options
*
* @see #runScriptOnNodesWithTag(String, org.jclouds.domain.Credentials,
* byte[], org.jclouds.compute.options.RunScriptOptions)
*/
Map<String, ExecResponse> runScriptOnNodesWithTag(String tag, Credentials credentials,
byte[] runScript);
/**
* Run the script on all nodes with the specific tag.
*
* @param tag tag to look up the nodes
* @param credentials credentials to use (same for all nodes)
* @param runScript script to run in byte format. If the script is a string, use
* {@link String#getBytes()} to retrieve the bytes
* @param options nullable options to how to run the script
* @return map with node identifiers and corresponding responses
*/
Map<String, ExecResponse> runScriptOnNodesWithTag(String tag, Credentials credentials,
byte[] runScript, RunScriptOptions options);
} }

View File

@ -27,6 +27,7 @@ import static com.google.common.base.Preconditions.checkArgument;
import static com.google.common.base.Preconditions.checkNotNull; import static com.google.common.base.Preconditions.checkNotNull;
import static org.jclouds.concurrent.ConcurrentUtils.awaitCompletion; import static org.jclouds.concurrent.ConcurrentUtils.awaitCompletion;
import static org.jclouds.concurrent.ConcurrentUtils.makeListenable; import static org.jclouds.concurrent.ConcurrentUtils.makeListenable;
import static org.jclouds.util.Utils.checkNotEmpty;
import java.util.Map; import java.util.Map;
import java.util.Set; import java.util.Set;
@ -34,6 +35,7 @@ import java.util.Map.Entry;
import java.util.concurrent.Callable; import java.util.concurrent.Callable;
import java.util.concurrent.ExecutorService; import java.util.concurrent.ExecutorService;
import javax.annotation.Nullable;
import javax.annotation.Resource; import javax.annotation.Resource;
import javax.inject.Inject; import javax.inject.Inject;
import javax.inject.Named; import javax.inject.Named;
@ -51,6 +53,7 @@ import org.jclouds.compute.domain.NodeState;
import org.jclouds.compute.domain.Size; import org.jclouds.compute.domain.Size;
import org.jclouds.compute.domain.Template; import org.jclouds.compute.domain.Template;
import org.jclouds.compute.domain.TemplateBuilder; import org.jclouds.compute.domain.TemplateBuilder;
import org.jclouds.compute.options.RunScriptOptions;
import org.jclouds.compute.reference.ComputeServiceConstants; import org.jclouds.compute.reference.ComputeServiceConstants;
import org.jclouds.compute.strategy.DestroyNodeStrategy; import org.jclouds.compute.strategy.DestroyNodeStrategy;
import org.jclouds.compute.strategy.GetNodeMetadataStrategy; import org.jclouds.compute.strategy.GetNodeMetadataStrategy;
@ -58,6 +61,7 @@ import org.jclouds.compute.strategy.ListNodesStrategy;
import org.jclouds.compute.strategy.RebootNodeStrategy; import org.jclouds.compute.strategy.RebootNodeStrategy;
import org.jclouds.compute.strategy.RunNodesAndAddToSetStrategy; import org.jclouds.compute.strategy.RunNodesAndAddToSetStrategy;
import org.jclouds.compute.util.ComputeUtils; import org.jclouds.compute.util.ComputeUtils;
import org.jclouds.domain.Credentials;
import org.jclouds.domain.Location; import org.jclouds.domain.Location;
import org.jclouds.logging.Logger; import org.jclouds.logging.Logger;
@ -68,6 +72,7 @@ import com.google.common.collect.Iterables;
import com.google.common.collect.Maps; import com.google.common.collect.Maps;
import com.google.common.collect.Sets; import com.google.common.collect.Sets;
import com.google.common.util.concurrent.ListenableFuture; import com.google.common.util.concurrent.ListenableFuture;
import org.jclouds.ssh.ExecResponse;
/** /**
* *
@ -76,244 +81,298 @@ import com.google.common.util.concurrent.ListenableFuture;
@Singleton @Singleton
public class BaseComputeService implements ComputeService { public class BaseComputeService implements ComputeService {
@Resource @Resource
@Named(ComputeServiceConstants.COMPUTE_LOGGER) @Named(ComputeServiceConstants.COMPUTE_LOGGER)
protected Logger logger = Logger.NULL; protected Logger logger = Logger.NULL;
protected final ComputeServiceContext context; protected final ComputeServiceContext context;
protected final Provider<Map<String, ? extends Image>> images; protected final Provider<Map<String, ? extends Image>> images;
protected final Provider<Map<String, ? extends Size>> sizes; protected final Provider<Map<String, ? extends Size>> sizes;
protected final Provider<Map<String, ? extends Location>> locations; protected final Provider<Map<String, ? extends Location>> locations;
protected final ListNodesStrategy listNodesStrategy; protected final ListNodesStrategy listNodesStrategy;
protected final GetNodeMetadataStrategy getNodeMetadataStrategy; protected final GetNodeMetadataStrategy getNodeMetadataStrategy;
protected final RunNodesAndAddToSetStrategy runNodesAndAddToSetStrategy; protected final RunNodesAndAddToSetStrategy runNodesAndAddToSetStrategy;
protected final RebootNodeStrategy rebootNodeStrategy; protected final RebootNodeStrategy rebootNodeStrategy;
protected final DestroyNodeStrategy destroyNodeStrategy; protected final DestroyNodeStrategy destroyNodeStrategy;
protected final Provider<TemplateBuilder> templateBuilderProvider; protected final Provider<TemplateBuilder> templateBuilderProvider;
protected final ComputeUtils utils; protected final ComputeUtils utils;
protected final ExecutorService executor; protected final ExecutorService executor;
private static class NodeMatchesTag implements Predicate<NodeMetadata> { private static class NodeMatchesTag implements Predicate<NodeMetadata> {
private final String tag; private final String tag;
public NodeMatchesTag(String tag) { public NodeMatchesTag(String tag) {
this.tag = tag; this.tag = tag;
} }
@Override @Override
public boolean apply(NodeMetadata from) { public boolean apply(NodeMetadata from) {
return from.getTag().equals(tag); return from.getTag().equals(tag);
} }
}; };
public static Function<ComputeMetadata, String> METADATA_TO_ID = new Function<ComputeMetadata, String>() { public static Function<ComputeMetadata, String> METADATA_TO_ID = new Function<ComputeMetadata, String>() {
@Override @Override
public String apply(ComputeMetadata from) { public String apply(ComputeMetadata from) {
return from.getId(); return from.getId();
} }
}; };
public static Function<ComputeMetadata, String> METADATA_TO_NAME = new Function<ComputeMetadata, String>() { public static Function<ComputeMetadata, String> METADATA_TO_NAME = new Function<ComputeMetadata, String>() {
@Override @Override
public String apply(ComputeMetadata from) { public String apply(ComputeMetadata from) {
return from.getName(); return from.getName();
} }
}; };
@Inject @Inject
protected BaseComputeService(ComputeServiceContext context, protected BaseComputeService(ComputeServiceContext context,
Provider<Map<String, ? extends Image>> images, Provider<Map<String, ? extends Image>> images,
Provider<Map<String, ? extends Size>> sizes, Provider<Map<String, ? extends Size>> sizes,
Provider<Map<String, ? extends Location>> locations, Provider<Map<String, ? extends Location>> locations,
ListNodesStrategy listNodesStrategy, GetNodeMetadataStrategy getNodeMetadataStrategy, ListNodesStrategy listNodesStrategy, GetNodeMetadataStrategy getNodeMetadataStrategy,
RunNodesAndAddToSetStrategy runNodesAndAddToSetStrategy, RunNodesAndAddToSetStrategy runNodesAndAddToSetStrategy,
RebootNodeStrategy rebootNodeStrategy, DestroyNodeStrategy destroyNodeStrategy, RebootNodeStrategy rebootNodeStrategy, DestroyNodeStrategy destroyNodeStrategy,
Provider<TemplateBuilder> templateBuilderProvider, ComputeUtils utils, Provider<TemplateBuilder> templateBuilderProvider, ComputeUtils utils,
@Named(Constants.PROPERTY_USER_THREADS) ExecutorService executor) { @Named(Constants.PROPERTY_USER_THREADS) ExecutorService executor) {
this.context = checkNotNull(context, "context"); this.context = checkNotNull(context, "context");
this.images = checkNotNull(images, "images"); this.images = checkNotNull(images, "images");
this.sizes = checkNotNull(sizes, "sizes"); this.sizes = checkNotNull(sizes, "sizes");
this.locations = checkNotNull(locations, "locations"); this.locations = checkNotNull(locations, "locations");
this.listNodesStrategy = checkNotNull(listNodesStrategy, "listNodesStrategy"); this.listNodesStrategy = checkNotNull(listNodesStrategy, "listNodesStrategy");
this.getNodeMetadataStrategy = checkNotNull(getNodeMetadataStrategy, this.getNodeMetadataStrategy = checkNotNull(getNodeMetadataStrategy,
"getNodeMetadataStrategy"); "getNodeMetadataStrategy");
this.runNodesAndAddToSetStrategy = checkNotNull(runNodesAndAddToSetStrategy, this.runNodesAndAddToSetStrategy = checkNotNull(runNodesAndAddToSetStrategy,
"runNodesAndAddToSetStrategy"); "runNodesAndAddToSetStrategy");
this.rebootNodeStrategy = checkNotNull(rebootNodeStrategy, "rebootNodeStrategy"); this.rebootNodeStrategy = checkNotNull(rebootNodeStrategy, "rebootNodeStrategy");
this.destroyNodeStrategy = checkNotNull(destroyNodeStrategy, "destroyNodeStrategy"); this.destroyNodeStrategy = checkNotNull(destroyNodeStrategy, "destroyNodeStrategy");
this.templateBuilderProvider = checkNotNull(templateBuilderProvider, this.templateBuilderProvider = checkNotNull(templateBuilderProvider,
"templateBuilderProvider"); "templateBuilderProvider");
this.utils = checkNotNull(utils, "utils"); this.utils = checkNotNull(utils, "utils");
this.executor = checkNotNull(executor, "executor"); this.executor = checkNotNull(executor, "executor");
} }
@Override @Override
public ComputeServiceContext getContext() { public ComputeServiceContext getContext() {
return context; return context;
} }
@Override @Override
public Map<String, ? extends NodeMetadata> runNodesWithTag(final String tag, int count, public Map<String, ? extends NodeMetadata> runNodesWithTag(final String tag, int count,
final Template template) { final Template template) {
checkArgument(tag.indexOf('-') == -1, "tag cannot contain hyphens"); checkArgument(tag.indexOf('-') == -1, "tag cannot contain hyphens");
checkNotNull(template.getLocation(), "location"); checkNotNull(template.getLocation(), "location");
logger.debug(">> running %d node%s tag(%s) location(%s) image(%s) size(%s) options(%s)", logger.debug(">> running %d node%s tag(%s) location(%s) image(%s) size(%s) options(%s)",
count, count > 1 ? "s" : "", tag, template.getLocation().getId(), template count, count > 1 ? "s" : "", tag, template.getLocation().getId(), template
.getImage().getId(), template.getSize().getId(), template.getOptions()); .getImage().getId(), template.getSize().getId(), template.getOptions());
final Set<NodeMetadata> nodes = Sets.newHashSet(); final Set<NodeMetadata> nodes = Sets.newHashSet();
Map<?, ListenableFuture<Void>> responses = runNodesAndAddToSetStrategy.execute(tag, count, Map<?, ListenableFuture<Void>> responses = runNodesAndAddToSetStrategy.execute(tag, count,
template, nodes); template, nodes);
Map<?, Exception> exceptions = awaitCompletion(responses, executor, null, logger, Map<?, Exception> exceptions = awaitCompletion(responses, executor, null, logger,
"starting nodes"); "starting nodes");
if (exceptions.size() > 0 && template.getOptions().shouldDestroyOnError()) { if (exceptions.size() > 0 && template.getOptions().shouldDestroyOnError()) {
ImmutableMap<?, ? extends ComputeMetadata> currentNodes = Maps.uniqueIndex( ImmutableMap<?, ? extends ComputeMetadata> currentNodes = Maps.uniqueIndex(
listNodesStrategy.execute(), METADATA_TO_ID); listNodesStrategy.execute(), METADATA_TO_ID);
for (Entry<?, Exception> entry : exceptions.entrySet()) { for (Entry<?, Exception> entry : exceptions.entrySet()) {
logger.error(entry.getValue(), "<< error applying nodes(%s) [%s] destroying ", entry logger.error(entry.getValue(), "<< error applying nodes(%s) [%s] destroying ", entry
.getKey(), entry.getValue().getMessage()); .getKey(), entry.getValue().getMessage());
destroyNode(currentNodes.get(entry.getKey())); destroyNode(currentNodes.get(entry.getKey()));
}
}
return Maps.uniqueIndex(nodes, METADATA_TO_ID);
}
@Override
public void destroyNode(ComputeMetadata node) {
checkArgument(node.getType() == ComputeType.NODE, "this is only valid for nodes, not "
+ node.getType());
checkNotNull(node.getId(), "node.id");
logger.debug(">> destroying node(%s)", node.getId());
boolean successful = destroyNodeStrategy.execute(node);
logger.debug("<< destroyed node(%s) success(%s)", node.getId(), successful);
}
@Override
public void destroyNodesWithTag(String tag) { // TODO parallel
logger.debug(">> destroying nodes by tag(%s)", tag);
Iterable<? extends NodeMetadata> nodesToDestroy = Iterables.filter(doGetNodesWithTag(tag)
.values(), new Predicate<NodeMetadata>() {
@Override
public boolean apply(NodeMetadata input) {
return input.getState() != NodeState.TERMINATED;
}
});
Map<NodeMetadata, ListenableFuture<Void>> responses = Maps.newHashMap();
for (final NodeMetadata node : nodesToDestroy) {
responses.put(node, makeListenable(executor.submit(new Callable<Void>() {
@Override
public Void call() throws Exception {
destroyNode(node);
return null;
} }
}), executor)); }
} return Maps.uniqueIndex(nodes, METADATA_TO_ID);
awaitCompletion(responses, executor, null, logger, "destroying nodes"); }
logger.debug("<< destroyed");
}
@Override @Override
public Map<String, ? extends ComputeMetadata> getNodes() { public void destroyNode(ComputeMetadata node) {
logger.debug(">> listing servers"); checkArgument(node.getType() == ComputeType.NODE, "this is only valid for nodes, not "
ImmutableMap<String, ? extends ComputeMetadata> map = Maps.uniqueIndex(listNodesStrategy + node.getType());
.execute(), METADATA_TO_ID); checkNotNull(node.getId(), "node.id");
logger.debug("<< list(%d)", map.size()); logger.debug(">> destroying node(%s)", node.getId());
return map; boolean successful = destroyNodeStrategy.execute(node);
} logger.debug("<< destroyed node(%s) success(%s)", node.getId(), successful);
}
/** @Override
* If the result of {@link ListNodesStrategy#execute} is a set of nodes, then return them. public void destroyNodesWithTag(String tag) { // TODO parallel
* Otherwise iteratively call {@link #getNodeMetadata} logger.debug(">> destroying nodes by tag(%s)", tag);
*/ Iterable<? extends NodeMetadata> nodesToDestroy = Iterables.filter(doGetNodesWithTag(tag)
protected Map<String, ? extends NodeMetadata> doGetNodesWithTag(final String tag) { .values(), new Predicate<NodeMetadata>() {
Iterable<? extends NodeMetadata> nodes = Iterables.filter(Iterables.transform(
listNodesStrategy.execute(), new Function<ComputeMetadata, NodeMetadata>() {
@Override
public NodeMetadata apply(ComputeMetadata from) {
return from instanceof NodeMetadata ? NodeMetadata.class.cast(from)
: getNodeMetadata(from);
}
}), new Predicate<NodeMetadata>() {
@Override
public boolean apply(NodeMetadata input) {
return tag.equals(input.getTag());
}
});
return Maps.uniqueIndex(Iterables.filter(nodes, new NodeMatchesTag(tag)), METADATA_TO_ID);
}
@Override
public Map<String, ? extends NodeMetadata> getNodesWithTag(String tag) {
logger.debug(">> listing nodes by tag(%s)", tag);
Map<String, ? extends NodeMetadata> nodes = doGetNodesWithTag(tag);
logger.debug("<< list(%d)", nodes.size());
return nodes;
}
@Override
public Map<String, ? extends Size> getSizes() {
return sizes.get();
}
@Override
public Map<String, ? extends Image> getImages() {
return images.get();
}
@Override
public Map<String, ? extends Location> getLocations() {
return locations.get();
}
@Override
public TemplateBuilder templateBuilder() {
return templateBuilderProvider.get();
}
@Override
public NodeMetadata getNodeMetadata(ComputeMetadata node) {
checkArgument(node.getType() == ComputeType.NODE, "this is only valid for nodes, not "
+ node.getType());
return getNodeMetadataStrategy.execute(node);
}
@Override
public void rebootNode(ComputeMetadata node) {
checkArgument(node.getType() == ComputeType.NODE, "this is only valid for nodes, not "
+ node.getType());
checkNotNull(node.getId(), "node.id");
logger.debug(">> rebooting node(%s)", node.getId());
boolean successful = rebootNodeStrategy.execute(node);
logger.debug("<< rebooted node(%s) success(%s)", node.getId(), successful);
}
@Override
public void rebootNodesWithTag(String tag) { // TODO parallel
logger.debug(">> rebooting nodes by tag(%s)", tag);
Iterable<? extends NodeMetadata> nodesToReboot = Iterables.filter(doGetNodesWithTag(tag)
.values(), new Predicate<NodeMetadata>() {
@Override
public boolean apply(NodeMetadata input) {
return input.getState() != NodeState.TERMINATED;
}
});
Map<NodeMetadata, ListenableFuture<Void>> responses = Maps.newHashMap();
for (final NodeMetadata node : nodesToReboot) {
responses.put(node, makeListenable(executor.submit(new Callable<Void>() {
@Override @Override
public Void call() throws Exception { public boolean apply(NodeMetadata input) {
rebootNode(node); return input.getState() != NodeState.TERMINATED;
return null;
} }
}), executor)); });
} Map<NodeMetadata, ListenableFuture<Void>> responses = Maps.newHashMap();
awaitCompletion(responses, executor, null, logger, "rebooting nodes"); for (final NodeMetadata node : nodesToDestroy) {
logger.debug("<< rebooted"); responses.put(node, makeListenable(executor.submit(new Callable<Void>() {
} @Override
public Void call() throws Exception {
destroyNode(node);
return null;
}
}), executor));
}
awaitCompletion(responses, executor, null, logger, "destroying nodes");
logger.debug("<< destroyed");
}
@Override
public Map<String, ? extends ComputeMetadata> getNodes() {
logger.debug(">> listing servers");
ImmutableMap<String, ? extends ComputeMetadata> map = Maps.uniqueIndex(listNodesStrategy
.execute(), METADATA_TO_ID);
logger.debug("<< list(%d)", map.size());
return map;
}
/**
* If the result of {@link ListNodesStrategy#execute} is a set of nodes, then return them.
* Otherwise iteratively call {@link #getNodeMetadata}
*/
protected Map<String, ? extends NodeMetadata> doGetNodesWithTag(final String tag) {
Iterable<? extends NodeMetadata> nodes = Iterables.filter(Iterables.transform(
listNodesStrategy.execute(), new Function<ComputeMetadata, NodeMetadata>() {
@Override
public NodeMetadata apply(ComputeMetadata from) {
return from instanceof NodeMetadata ? NodeMetadata.class.cast(from)
: getNodeMetadata(from);
}
}), new Predicate<NodeMetadata>() {
@Override
public boolean apply(NodeMetadata input) {
return tag.equals(input.getTag());
}
});
return Maps.uniqueIndex(Iterables.filter(nodes, new NodeMatchesTag(tag)), METADATA_TO_ID);
}
@Override
public Map<String, ? extends NodeMetadata> getNodesWithTag(String tag) {
logger.debug(">> listing nodes by tag(%s)", tag);
Map<String, ? extends NodeMetadata> nodes = doGetNodesWithTag(tag);
logger.debug("<< list(%d)", nodes.size());
return nodes;
}
@Override
public Map<String, ? extends Size> getSizes() {
return sizes.get();
}
@Override
public Map<String, ? extends Image> getImages() {
return images.get();
}
@Override
public Map<String, ? extends Location> getLocations() {
return locations.get();
}
@Override
public TemplateBuilder templateBuilder() {
return templateBuilderProvider.get();
}
@Override
public NodeMetadata getNodeMetadata(ComputeMetadata node) {
checkArgument(node.getType() == ComputeType.NODE, "this is only valid for nodes, not "
+ node.getType());
return getNodeMetadataStrategy.execute(node);
}
@Override
public void rebootNode(ComputeMetadata node) {
checkArgument(node.getType() == ComputeType.NODE, "this is only valid for nodes, not "
+ node.getType());
checkNotNull(node.getId(), "node.id");
logger.debug(">> rebooting node(%s)", node.getId());
boolean successful = rebootNodeStrategy.execute(node);
logger.debug("<< rebooted node(%s) success(%s)", node.getId(), successful);
}
@Override
public void rebootNodesWithTag(String tag) { // TODO parallel
logger.debug(">> rebooting nodes by tag(%s)", tag);
Iterable<? extends NodeMetadata> nodesToReboot = Iterables.filter(doGetNodesWithTag(tag)
.values(), new Predicate<NodeMetadata>() {
@Override
public boolean apply(NodeMetadata input) {
return input.getState() != NodeState.TERMINATED;
}
});
Map<NodeMetadata, ListenableFuture<Void>> responses = Maps.newHashMap();
for (final NodeMetadata node : nodesToReboot) {
responses.put(node, makeListenable(executor.submit(new Callable<Void>() {
@Override
public Void call() throws Exception {
rebootNode(node);
return null;
}
}), executor));
}
awaitCompletion(responses, executor, null, logger, "rebooting nodes");
logger.debug("<< rebooted");
}
/**
* @see #runScriptOnNodesWithTag(String, org.jclouds.domain.Credentials, byte[],
* org.jclouds.compute.options.RunScriptOptions)
*/
public Map<String, ExecResponse> runScriptOnNodesWithTag(String tag, Credentials credentials,
byte[] runScript) {
return runScriptOnNodesWithTag(tag, credentials, runScript, RunScriptOptions.NONE);
}
/**
* Run the script on all nodes with the specific tag.
*
* @param tag tag to look up the nodes
* @param credentials nullable credentials to use (same for all nodes).
* @param runScript script to run in byte format. If the script is a string, use
* {@link String#getBytes()} to retrieve the bytes
* @param options nullable options to how to run the script
* @return map with node identifiers and corresponding responses
*/
public Map<String, ExecResponse> runScriptOnNodesWithTag(String tag, @Nullable Credentials credentials,
byte[] runScript, @Nullable RunScriptOptions options) {
checkNotEmpty(tag, "Tag must be provided");
checkNotNull(runScript,
"The script (represented by bytes array - use \"script\".getBytes() must be provided");
if(options == null) options = RunScriptOptions.NONE;
Map<String, ? extends NodeMetadata> nodes = getNodesWithTag(tag);
Map<String, ExecResponse> responses = Maps.newHashMap();
for(NodeMetadata node : nodes.values()) {
if(NodeState.RUNNING != node.getState()) continue; //make sure the node is active
if(options.isOverrideCredentials()) {
//override the credentials with provided to this method
checkNotNull(credentials, "If the credentials need to be overridden, they can't be null");
node = ComputeUtils.installNewCredentials(node, credentials);
} else {
//don't override
checkNotNull(node.getCredentials(), "If the default credentials need to be used, they can't be null");
}
//todo: execute script as root if required
ComputeUtils.SshCallable<?> callable = utils.runScriptOnNode(node, "computeserv.sh", runScript);
Map<ComputeUtils.SshCallable<?>, ?> scriptRunResults = utils.runCallablesOnNode(node,
Sets.newHashSet(callable),
null);
responses.put(node.getId(),
(ExecResponse) scriptRunResults.get(callable));
}
return responses;
}
} }

View File

@ -0,0 +1,88 @@
/**
*
* Copyright (C) 2010 Cloud Conscious, LLC. <info@cloudconscious.com>
*
* ====================================================================
* Licensed 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.jclouds.compute.options;
/**
* Enables additional options for running a script.
*
* @author Oleksiy Yarmula
*/
public class RunScriptOptions {
/**
* Default options. The default settings are:
* <ul>
* <li>override the credentials with ones supplied in
* call to {@link org.jclouds.compute.ComputeService#runScriptOnNodesWithTag}</li>
* <li>do not run the script as root (run with current privileges)</li>
* </ul>
*/
public static final RunScriptOptions NONE = new RunScriptOptions();
private boolean overrideCredentials = true;
private boolean runAsRoot = false;
private void overrideCredentials(boolean overrideCredentials) {
this.overrideCredentials = overrideCredentials;
}
private void runAsRoot(boolean runAsRoot) {
this.runAsRoot = runAsRoot;
}
/**
* Whether to override the credentials with ones supplied in
* call to {@link org.jclouds.compute.ComputeService#runScriptOnNodesWithTag}.
* By default, true.
* @return value
*/
public boolean isOverrideCredentials() {
return overrideCredentials;
}
/**
* Whether to run the script as root (run with current privileges).
* By default, false.
* @return value
*/
public boolean isRunAsRoot() {
return runAsRoot;
}
public static class Builder {
private RunScriptOptions options;
public Builder overrideCredentials(boolean value) {
if(options == null) options = new RunScriptOptions();
options.overrideCredentials(value);
return this;
}
public Builder runAsRoot(boolean value) {
if(options == null) options = new RunScriptOptions();
options.runAsRoot(value);
return this;
}
public RunScriptOptions build() {
return options;
}
}
}

View File

@ -30,6 +30,7 @@ import java.util.Comparator;
import java.util.List; import java.util.List;
import java.util.Map; import java.util.Map;
import java.util.concurrent.Callable; import java.util.concurrent.Callable;
import java.util.concurrent.ExecutionException;
import java.util.concurrent.ExecutorService; import java.util.concurrent.ExecutorService;
import javax.annotation.Nullable; import javax.annotation.Nullable;
@ -39,9 +40,11 @@ import javax.inject.Named;
import org.jclouds.Constants; import org.jclouds.Constants;
import org.jclouds.compute.domain.ComputeMetadata; import org.jclouds.compute.domain.ComputeMetadata;
import org.jclouds.compute.domain.NodeMetadata; import org.jclouds.compute.domain.NodeMetadata;
import org.jclouds.compute.domain.internal.NodeMetadataImpl;
import org.jclouds.compute.options.TemplateOptions; import org.jclouds.compute.options.TemplateOptions;
import org.jclouds.compute.reference.ComputeServiceConstants; import org.jclouds.compute.reference.ComputeServiceConstants;
import org.jclouds.concurrent.ConcurrentUtils; import org.jclouds.concurrent.ConcurrentUtils;
import org.jclouds.domain.Credentials;
import org.jclouds.logging.Logger; import org.jclouds.logging.Logger;
import org.jclouds.scriptbuilder.InitBuilder; import org.jclouds.scriptbuilder.InitBuilder;
import org.jclouds.scriptbuilder.domain.OsFamily; import org.jclouds.scriptbuilder.domain.OsFamily;
@ -63,243 +66,269 @@ import com.google.inject.Inject;
* @author Adrian Cole * @author Adrian Cole
*/ */
public class ComputeUtils { public class ComputeUtils {
@Resource @Resource
@Named(ComputeServiceConstants.COMPUTE_LOGGER) @Named(ComputeServiceConstants.COMPUTE_LOGGER)
protected Logger logger = Logger.NULL; protected Logger logger = Logger.NULL;
@Inject(optional = true) @Inject(optional = true)
private SshClient.Factory sshFactory; private SshClient.Factory sshFactory;
protected final Predicate<SshClient> runScriptNotRunning; protected final Predicate<SshClient> runScriptNotRunning;
private final Predicate<InetSocketAddress> socketTester; private final Predicate<InetSocketAddress> socketTester;
private final ExecutorService executor; private final ExecutorService executor;
@Inject @Inject
public ComputeUtils(Predicate<InetSocketAddress> socketTester, public ComputeUtils(Predicate<InetSocketAddress> socketTester,
@Named("NOT_RUNNING") Predicate<SshClient> runScriptNotRunning, @Named("NOT_RUNNING") Predicate<SshClient> runScriptNotRunning,
@Named(Constants.PROPERTY_USER_THREADS) ExecutorService executor) { @Named(Constants.PROPERTY_USER_THREADS) ExecutorService executor) {
this.socketTester = socketTester; this.socketTester = socketTester;
this.runScriptNotRunning = runScriptNotRunning; this.runScriptNotRunning = runScriptNotRunning;
this.executor = executor; this.executor = executor;
} }
public static Iterable<? extends ComputeMetadata> filterByName( public static Iterable<? extends ComputeMetadata> filterByName(
Iterable<? extends ComputeMetadata> nodes, final String name) { Iterable<? extends ComputeMetadata> nodes, final String name) {
return Iterables.filter(nodes, new Predicate<ComputeMetadata>() { return Iterables.filter(nodes, new Predicate<ComputeMetadata>() {
@Override @Override
public boolean apply(ComputeMetadata input) { public boolean apply(ComputeMetadata input) {
return input.getName().equalsIgnoreCase(name); return input.getName().equalsIgnoreCase(name);
}
});
}
public static final Comparator<InetAddress> ADDRESS_COMPARATOR = new Comparator<InetAddress>() {
@Override
public int compare(InetAddress o1, InetAddress o2) {
return (o1 == o2) ? 0 : o1.getHostAddress().compareTo(o2.getHostAddress());
}
};
public void runOptionsOnNode(NodeMetadata node, TemplateOptions options) {
List<SshCallable<?>> callables = Lists.newArrayList();
if (options.getRunScript() != null) {
callables.add(runScriptOnNode(node, "runscript.sh", options.getRunScript()));
}
if (options.getPublicKey() != null) {
callables.add(authorizeKeyOnNode(node, options.getPublicKey()));
}
// changing the key "MUST" come last or else the other commands may fail.
if (callables.size() > 0 || options.getPrivateKey() != null) {
runCallablesOnNode(node, callables, options.getPrivateKey() != null ? installKeyOnNode(
node, options.getPrivateKey()) : null);
}
}
public InstallRSAPrivateKey installKeyOnNode(NodeMetadata node, String privateKey) {
return new InstallRSAPrivateKey(node, privateKey);
}
public AuthorizeRSAPublicKey authorizeKeyOnNode(NodeMetadata node, String publicKey) {
return new AuthorizeRSAPublicKey(node, publicKey);
}
public RunScriptOnNode runScriptOnNode(NodeMetadata node, String scriptName, byte[] script) {
return new RunScriptOnNode(runScriptNotRunning, node, scriptName, script);
}
public void runCallablesOnNode(NodeMetadata node, Iterable<? extends SshCallable<?>> parallel,
@Nullable SshCallable<?> last) {
checkState(this.sshFactory != null, "runScript requested, but no SshModule configured");
InetSocketAddress socket = new InetSocketAddress(Iterables.get(node.getPublicAddresses(), 0),
22);
socketTester.apply(socket);
SshClient ssh = isKeyAuth(node) ? sshFactory.create(socket, node.getCredentials().account,
node.getCredentials().key.getBytes()) : sshFactory.create(socket, node
.getCredentials().account, node.getCredentials().key);
for (int i = 0; i < 3; i++) {
try {
ssh.connect();
Map<SshCallable<?>, ListenableFuture<?>> responses = Maps.newHashMap();
for (SshCallable<?> callable : parallel) {
callable.setConnection(ssh, logger);
responses.put(callable, ConcurrentUtils.makeListenable(executor.submit(callable),
executor));
} }
});
}
Map<SshCallable<?>, Exception> exceptions = awaitCompletion(responses, executor, null, public static final Comparator<InetAddress> ADDRESS_COMPARATOR = new Comparator<InetAddress>() {
logger, "ssh");
if (exceptions.size() > 0) @Override
throw new RuntimeException(String.format("error invoking callables on host %s: %s", public int compare(InetAddress o1, InetAddress o2) {
socket, exceptions)); return (o1 == o2) ? 0 : o1.getHostAddress().compareTo(o2.getHostAddress());
if (last != null) { }
last.setConnection(ssh, logger);
try { };
last.call();
} catch (Exception e) { public void runOptionsOnNode(NodeMetadata node, TemplateOptions options) {
Throwables.propagate(e); List<SshCallable<?>> callables = Lists.newArrayList();
} if (options.getRunScript() != null) {
callables.add(runScriptOnNode(node, "runscript.sh", options.getRunScript()));
}
if (options.getPublicKey() != null) {
callables.add(authorizeKeyOnNode(node, options.getPublicKey()));
}
// changing the key "MUST" come last or else the other commands may fail.
if (callables.size() > 0 || options.getPrivateKey() != null) {
runCallablesOnNode(node, callables, options.getPrivateKey() != null ? installKeyOnNode(
node, options.getPrivateKey()) : null);
}
}
public InstallRSAPrivateKey installKeyOnNode(NodeMetadata node, String privateKey) {
return new InstallRSAPrivateKey(node, privateKey);
}
public AuthorizeRSAPublicKey authorizeKeyOnNode(NodeMetadata node, String publicKey) {
return new AuthorizeRSAPublicKey(node, publicKey);
}
public RunScriptOnNode runScriptOnNode(NodeMetadata node, String scriptName, byte[] script) {
return new RunScriptOnNode(runScriptNotRunning, node, scriptName, script);
}
public Map<SshCallable<?>, ?> runCallablesOnNode(NodeMetadata node, Iterable<? extends SshCallable<?>> parallel,
@Nullable SshCallable<?> last) {
checkState(this.sshFactory != null, "runScript requested, but no SshModule configured");
InetSocketAddress socket = new InetSocketAddress(Iterables.get(node.getPublicAddresses(), 0),
22);
socketTester.apply(socket);
SshClient ssh = isKeyAuth(node) ? sshFactory.create(socket, node.getCredentials().account,
node.getCredentials().key.getBytes()) : sshFactory.create(socket, node
.getCredentials().account, node.getCredentials().key);
for (int i = 0; i < 3; i++) {
try {
ssh.connect();
Map<SshCallable<?>, ListenableFuture<?>> responses = Maps.newHashMap();
for (SshCallable<?> callable : parallel) {
callable.setConnection(ssh, logger);
responses.put(callable, ConcurrentUtils.makeListenable(executor.submit(callable),
executor));
}
Map<SshCallable<?>, Exception> exceptions = awaitCompletion(responses, executor, null,
logger, "ssh");
if (exceptions.size() > 0)
throw new RuntimeException(String.format("error invoking callables on host %s: %s",
socket, exceptions));
if (last != null) {
last.setConnection(ssh, logger);
try {
last.call();
} catch (Exception e) {
Throwables.propagate(e);
}
}
return transform(responses);
} catch (RuntimeException from) {
if (Iterables.size(Iterables.filter(Throwables.getCausalChain(from),
ConnectException.class)) >= 1// auth fail sometimes happens in EC2
|| Throwables.getRootCause(from).getMessage().indexOf("Auth fail") != -1
|| Throwables.getRootCause(from).getMessage().indexOf("invalid privatekey") != -1) {
try {
Thread.sleep(100);
} catch (InterruptedException e) {
throw Throwables.propagate(e);
}
continue;
}
throw Throwables.propagate(from);
} finally {
if (ssh != null)
ssh.disconnect();
} }
break; }
} catch (RuntimeException from) { throw new RuntimeException(String.format("Couldn't connect to node %s and run the script", node.getId()));
if (Iterables.size(Iterables.filter(Throwables.getCausalChain(from), }
ConnectException.class)) >= 1// auth fail sometimes happens in EC2
|| Throwables.getRootCause(from).getMessage().indexOf("Auth fail") != -1 public <T> Map<SshCallable<?>, T> transform(Map<SshCallable<?>, ListenableFuture<?>> responses) {
|| Throwables.getRootCause(from).getMessage().indexOf("invalid privatekey") != -1) { Map<SshCallable<?>, T> actualResponses = Maps.newHashMap();
try { for(Map.Entry<SshCallable<?>, ListenableFuture<?>> entry : responses.entrySet()) {
Thread.sleep(100); try {
} catch (InterruptedException e) { actualResponses.put(entry.getKey(), (T) entry.getValue().get());
Throwables.propagate(e); } catch(InterruptedException e) {
} throw Throwables.propagate(e);
continue; } catch(ExecutionException e) {
throw Throwables.propagate(e);
} }
Throwables.propagate(from); }
} finally { return actualResponses;
if (ssh != null) }
ssh.disconnect();
}
}
}
public static interface SshCallable<T> extends Callable<T> { public static interface SshCallable<T> extends Callable<T> {
void setConnection(SshClient ssh, Logger logger); void setConnection(SshClient ssh, Logger logger);
} }
public static class RunScriptOnNode implements SshCallable<ExecResponse> { public static class RunScriptOnNode implements SshCallable<ExecResponse> {
private SshClient ssh; private SshClient ssh;
protected final Predicate<SshClient> runScriptNotRunning; protected final Predicate<SshClient> runScriptNotRunning;
private final NodeMetadata node; private final NodeMetadata node;
private final String scriptName; private final String scriptName;
private final byte[] script; private final byte[] script;
private Logger logger = Logger.NULL; private Logger logger = Logger.NULL;
RunScriptOnNode(@Named("NOT_RUNNING") Predicate<SshClient> runScriptNotRunning, RunScriptOnNode(@Named("NOT_RUNNING") Predicate<SshClient> runScriptNotRunning,
NodeMetadata node, String scriptName, byte[] script) { NodeMetadata node, String scriptName, byte[] script) {
this.runScriptNotRunning = runScriptNotRunning; this.runScriptNotRunning = runScriptNotRunning;
this.node = checkNotNull(node, "node"); this.node = checkNotNull(node, "node");
this.scriptName = checkNotNull(scriptName, "scriptName"); this.scriptName = checkNotNull(scriptName, "scriptName");
this.script = new InitBuilder("runscript", "/tmp", "/tmp", ImmutableMap this.script = new InitBuilder("runscript", "/tmp", "/tmp", ImmutableMap
.<String, String> of(), Iterables.toArray(Splitter.on("\n").split( .<String, String> of(), Iterables.toArray(Splitter.on("\n").split(
new String(checkNotNull(script, "script"))), String.class)).build(OsFamily.UNIX) new String(checkNotNull(script, "script"))), String.class)).build(OsFamily.UNIX)
.getBytes(); .getBytes();
} }
@Override @Override
public ExecResponse call() throws Exception { public ExecResponse call() throws Exception {
ssh.put(scriptName, new ByteArrayInputStream(script)); ssh.put(scriptName, new ByteArrayInputStream(script));
ExecResponse returnVal = ssh.exec("chmod 755 " + scriptName); ExecResponse returnVal = ssh.exec("chmod 755 " + scriptName);
returnVal = ssh.exec("./" + scriptName + " init"); returnVal = ssh.exec("./" + scriptName + " init");
if (node.getCredentials().account.equals("root")) { if (node.getCredentials().account.equals("root")) {
logger.debug(">> running %s as %s@%s", scriptName, node.getCredentials().account, logger.debug(">> running %s as %s@%s", scriptName, node.getCredentials().account,
Iterables.get(node.getPublicAddresses(), 0).getHostAddress()); Iterables.get(node.getPublicAddresses(), 0).getHostAddress());
returnVal = ssh.exec("./" + scriptName + " start"); returnVal = ssh.exec("./" + scriptName + " start");
} else if (isKeyAuth(node)) { } else if (isKeyAuth(node)) {
logger.debug(">> running sudo %s as %s@%s", scriptName, node.getCredentials().account, logger.debug(">> running sudo %s as %s@%s", scriptName, node.getCredentials().account,
Iterables.get(node.getPublicAddresses(), 0).getHostAddress()); Iterables.get(node.getPublicAddresses(), 0).getHostAddress());
returnVal = ssh.exec("sudo ./" + scriptName + " start"); returnVal = ssh.exec("sudo ./" + scriptName + " start");
} else { } else {
logger.debug(">> running sudo -S %s as %s@%s", scriptName, logger.debug(">> running sudo -S %s as %s@%s", scriptName,
node.getCredentials().account, Iterables.get(node.getPublicAddresses(), 0) node.getCredentials().account, Iterables.get(node.getPublicAddresses(), 0)
.getHostAddress()); .getHostAddress());
returnVal = ssh.exec(String.format("echo %s|sudo -S ./%s", node.getCredentials().key, returnVal = ssh.exec(String.format("echo %s|sudo -S ./%s", node.getCredentials().key,
scriptName + " start")); scriptName + " start"));
} }
runScriptNotRunning.apply(ssh); runScriptNotRunning.apply(ssh);
logger.debug("<< complete(%d)", returnVal.getExitCode()); logger.debug("<< complete(%d)", returnVal.getExitCode());
return returnVal; return returnVal;
} }
@Override @Override
public void setConnection(SshClient ssh, Logger logger) { public void setConnection(SshClient ssh, Logger logger) {
this.logger = checkNotNull(logger, "logger"); this.logger = checkNotNull(logger, "logger");
this.ssh = checkNotNull(ssh, "ssh"); this.ssh = checkNotNull(ssh, "ssh");
} }
} }
public static class InstallRSAPrivateKey implements SshCallable<ExecResponse> { public static class InstallRSAPrivateKey implements SshCallable<ExecResponse> {
private SshClient ssh; private SshClient ssh;
private final NodeMetadata node; private final NodeMetadata node;
private final String privateKey; private final String privateKey;
private Logger logger = Logger.NULL; private Logger logger = Logger.NULL;
InstallRSAPrivateKey(NodeMetadata node, String privateKey) { InstallRSAPrivateKey(NodeMetadata node, String privateKey) {
this.node = checkNotNull(node, "node"); this.node = checkNotNull(node, "node");
this.privateKey = checkNotNull(privateKey, "privateKey"); this.privateKey = checkNotNull(privateKey, "privateKey");
} }
@Override @Override
public ExecResponse call() throws Exception { public ExecResponse call() throws Exception {
ssh.exec("mkdir .ssh"); ssh.exec("mkdir .ssh");
ssh.put(".ssh/id_rsa", new ByteArrayInputStream(privateKey.getBytes())); ssh.put(".ssh/id_rsa", new ByteArrayInputStream(privateKey.getBytes()));
logger.debug(">> installing rsa key for %s@%s", node.getCredentials().account, Iterables logger.debug(">> installing rsa key for %s@%s", node.getCredentials().account, Iterables
.get(node.getPublicAddresses(), 0).getHostAddress()); .get(node.getPublicAddresses(), 0).getHostAddress());
return ssh.exec("chmod 600 .ssh/id_rsa"); return ssh.exec("chmod 600 .ssh/id_rsa");
} }
@Override @Override
public void setConnection(SshClient ssh, Logger logger) { public void setConnection(SshClient ssh, Logger logger) {
this.logger = checkNotNull(logger, "logger"); this.logger = checkNotNull(logger, "logger");
this.ssh = checkNotNull(ssh, "ssh"); this.ssh = checkNotNull(ssh, "ssh");
} }
} }
public static class AuthorizeRSAPublicKey implements SshCallable<ExecResponse> { public static class AuthorizeRSAPublicKey implements SshCallable<ExecResponse> {
private SshClient ssh; private SshClient ssh;
private final NodeMetadata node; private final NodeMetadata node;
private final String publicKey; private final String publicKey;
private Logger logger = Logger.NULL; private Logger logger = Logger.NULL;
AuthorizeRSAPublicKey(NodeMetadata node, String publicKey) { AuthorizeRSAPublicKey(NodeMetadata node, String publicKey) {
this.node = checkNotNull(node, "node"); this.node = checkNotNull(node, "node");
this.publicKey = checkNotNull(publicKey, "publicKey"); this.publicKey = checkNotNull(publicKey, "publicKey");
} }
@Override @Override
public ExecResponse call() throws Exception { public ExecResponse call() throws Exception {
ssh.exec("mkdir .ssh"); ssh.exec("mkdir .ssh");
ssh.put(".ssh/id_rsa.pub", new ByteArrayInputStream(publicKey.getBytes())); ssh.put(".ssh/id_rsa.pub", new ByteArrayInputStream(publicKey.getBytes()));
logger.debug(">> authorizing rsa public key for %s@%s", node.getCredentials().account, logger.debug(">> authorizing rsa public key for %s@%s", node.getCredentials().account,
Iterables.get(node.getPublicAddresses(), 0).getHostAddress()); Iterables.get(node.getPublicAddresses(), 0).getHostAddress());
ExecResponse returnVal = ssh.exec("cat .ssh/id_rsa.pub >> .ssh/authorized_keys"); ExecResponse returnVal = ssh.exec("cat .ssh/id_rsa.pub >> .ssh/authorized_keys");
returnVal = ssh.exec("chmod 600 .ssh/authorized_keys"); returnVal = ssh.exec("chmod 600 .ssh/authorized_keys");
logger.debug("<< complete(%d)", returnVal.getExitCode()); logger.debug("<< complete(%d)", returnVal.getExitCode());
return returnVal; return returnVal;
} }
@Override @Override
public void setConnection(SshClient ssh, Logger logger) { public void setConnection(SshClient ssh, Logger logger) {
this.logger = checkNotNull(logger, "logger"); this.logger = checkNotNull(logger, "logger");
this.ssh = checkNotNull(ssh, "ssh"); this.ssh = checkNotNull(ssh, "ssh");
} }
} }
public static boolean isKeyAuth(NodeMetadata createdNode) { public static boolean isKeyAuth(NodeMetadata createdNode) {
return createdNode.getCredentials().key != null return createdNode.getCredentials().key != null
&& createdNode.getCredentials().key.startsWith("-----BEGIN RSA PRIVATE KEY-----"); && createdNode.getCredentials().key.startsWith("-----BEGIN RSA PRIVATE KEY-----");
} }
/**
* Given the instances of {@link NodeMetadata} (immutable)
* and {@link Credentials} (immutable), returns a new instance of {@link NodeMetadata}
* that has new credentials
*/
public static NodeMetadata installNewCredentials(NodeMetadata node, Credentials newCredentials) {
return new NodeMetadataImpl(node.getId(), node.getName(), node.getLocationId(), node.getUri(),
node.getUserMetadata(), node.getTag(), node.getState(), node. getPublicAddresses(),
node.getPrivateAddresses(), node.getExtra(), newCredentials);
}
} }

View File

@ -18,6 +18,7 @@
*/ */
package org.jclouds.util; package org.jclouds.util;
import static com.google.common.base.Preconditions.checkArgument;
import static com.google.common.base.Preconditions.checkNotNull; import static com.google.common.base.Preconditions.checkNotNull;
import static org.jclouds.util.Patterns.CHAR_TO_PATTERN; import static org.jclouds.util.Patterns.CHAR_TO_PATTERN;
import static org.jclouds.util.Patterns.TOKEN_TO_PATTERN; import static org.jclouds.util.Patterns.TOKEN_TO_PATTERN;
@ -49,156 +50,177 @@ import com.google.common.io.OutputSupplier;
* @author Adrian Cole * @author Adrian Cole
*/ */
public class Utils { public class Utils {
public static final String UTF8_ENCODING = "UTF-8"; public static final String UTF8_ENCODING = "UTF-8";
public static Object propagateOrNull(Exception from) { public static Object propagateOrNull(Exception from) {
Throwables.propagate(from); Throwables.propagate(from);
assert false : "exception should have propogated"; assert false : "exception should have propogated";
return null; return null;
} }
public static String replaceTokens(String value, Collection<Entry<String, String>> tokenValues) { public static String replaceTokens(String value, Collection<Entry<String, String>> tokenValues) {
for (Entry<String, String> tokenValue : tokenValues) { for (Entry<String, String> tokenValue : tokenValues) {
value = replaceAll(value, TOKEN_TO_PATTERN.get(tokenValue.getKey()), tokenValue.getValue()); value = replaceAll(value, TOKEN_TO_PATTERN.get(tokenValue.getKey()), tokenValue.getValue());
} }
return value; return value;
} }
public static String replaceAll(String returnVal, Pattern pattern, String replace) { public static String replaceAll(String returnVal, Pattern pattern, String replace) {
Matcher m = pattern.matcher(returnVal); Matcher m = pattern.matcher(returnVal);
returnVal = m.replaceAll(replace); returnVal = m.replaceAll(replace);
return returnVal; return returnVal;
} }
public static String replaceAll(String input, char ifMatch, Pattern pattern, String replacement) { public static String replaceAll(String input, char ifMatch, Pattern pattern, String replacement) {
if (input.indexOf(ifMatch) != -1) { if (input.indexOf(ifMatch) != -1) {
input = pattern.matcher(input).replaceAll(replacement); input = pattern.matcher(input).replaceAll(replacement);
} }
return input; return input;
} }
public static String replaceAll(String input, char match, String replacement) { public static String replaceAll(String input, char match, String replacement) {
if (input.indexOf(match) != -1) { if (input.indexOf(match) != -1) {
input = CHAR_TO_PATTERN.get(match).matcher(input).replaceAll(replacement); input = CHAR_TO_PATTERN.get(match).matcher(input).replaceAll(replacement);
} }
return input; return input;
} }
/** /**
* converts an {@link OutputStream} to an {@link OutputSupplier} * converts an {@link OutputStream} to an {@link OutputSupplier}
* *
*/ */
public static OutputSupplier<OutputStream> newOutputStreamSupplier(final OutputStream output) { public static OutputSupplier<OutputStream> newOutputStreamSupplier(final OutputStream output) {
checkNotNull(output, "output"); checkNotNull(output, "output");
return new OutputSupplier<OutputStream>() { return new OutputSupplier<OutputStream>() {
public OutputStream getOutput() throws IOException { public OutputStream getOutput() throws IOException {
return output; return output;
} }
}; };
} }
public static boolean enventuallyTrue(Supplier<Boolean> assertion, long inconsistencyMillis) public static boolean enventuallyTrue(Supplier<Boolean> assertion, long inconsistencyMillis)
throws InterruptedException { throws InterruptedException {
for (int i = 0; i < 30; i++) { for (int i = 0; i < 30; i++) {
if (!assertion.get()) { if (!assertion.get()) {
Thread.sleep(inconsistencyMillis / 30); Thread.sleep(inconsistencyMillis / 30);
continue; continue;
} }
return true; return true;
} }
return false; return false;
} }
@Resource @Resource
protected static Logger logger = Logger.NULL; protected static Logger logger = Logger.NULL;
public static String toStringAndClose(InputStream input) throws IOException { public static String toStringAndClose(InputStream input) throws IOException {
checkNotNull(input, "input"); checkNotNull(input, "input");
try { try {
return new String(ByteStreams.toByteArray(input), Charsets.UTF_8); return new String(ByteStreams.toByteArray(input), Charsets.UTF_8);
} catch (IOException e) { } catch (IOException e) {
logger.warn(e, "Failed to read from stream"); logger.warn(e, "Failed to read from stream");
return null; return null;
} catch (NullPointerException e) { } catch (NullPointerException e) {
return null; return null;
} finally { } finally {
Closeables.closeQuietly(input); Closeables.closeQuietly(input);
} }
} }
public static InputStream toInputStream(String in) { public static InputStream toInputStream(String in) {
try { try {
return ByteStreams.newInputStreamSupplier(in.getBytes(Charsets.UTF_8)).getInput(); return ByteStreams.newInputStreamSupplier(in.getBytes(Charsets.UTF_8)).getInput();
} catch (IOException e) { } catch (IOException e) {
logger.warn(e, "Failed to convert %s to an inputStream", in); logger.warn(e, "Failed to convert %s to an inputStream", in);
throw new RuntimeException(e); throw new RuntimeException(e);
} }
} }
/** /**
* Encode the given string with the given encoding, if possible. If the encoding fails with * Encode the given string with the given encoding, if possible. If the encoding fails with
* {@link UnsupportedEncodingException}, log a warning and fall back to the system's default * {@link UnsupportedEncodingException}, log a warning and fall back to the system's default
* encoding. * encoding.
* *
* @param str * @param str
* what to encode * what to encode
* @param charsetName * @param charsetName
* the name of a supported {@link java.nio.charset.Charset </code>charset<code>} * the name of a supported {@link java.nio.charset.Charset </code>charset<code>}
* @return properly encoded String. * @return properly encoded String.
*/ */
public static byte[] encodeString(String str, String charsetName) { public static byte[] encodeString(String str, String charsetName) {
try { try {
return str.getBytes(charsetName); return str.getBytes(charsetName);
} catch (UnsupportedEncodingException e) { } catch (UnsupportedEncodingException e) {
logger.warn(e, "Failed to encode string to bytes with encoding " + charsetName logger.warn(e, "Failed to encode string to bytes with encoding " + charsetName
+ ". Falling back to system's default encoding"); + ". Falling back to system's default encoding");
return str.getBytes(); return str.getBytes();
} }
} }
/** /**
* Encode the given string with the UTF-8 encoding, the sane default. In the very unlikely event * Encode the given string with the UTF-8 encoding, the sane default. In the very unlikely event
* the encoding fails with {@link UnsupportedEncodingException}, log a warning and fall back to * the encoding fails with {@link UnsupportedEncodingException}, log a warning and fall back to
* the system's default encoding. * the system's default encoding.
* *
* @param str * @param str
* what to encode * what to encode
* @return properly encoded String. * @return properly encoded String.
*/ */
public static byte[] encodeString(String str) { public static byte[] encodeString(String str) {
return encodeString(str, UTF8_ENCODING); return encodeString(str, UTF8_ENCODING);
} }
/** /**
* replaces tokens that are expressed as <code>{token}</code> * replaces tokens that are expressed as <code>{token}</code>
* *
* <p/> * <p/>
* ex. if input is "hello {where}"<br/> * ex. if input is "hello {where}"<br/>
* and replacements is "where" -> "world" <br/> * and replacements is "where" -> "world" <br/>
* then replaceTokens returns "hello world" * then replaceTokens returns "hello world"
* *
* @param input * @param input
* source to replace * source to replace
* @param replacements * @param replacements
* token/value pairs * token/value pairs
*/ */
public static String replaceTokens(String input, Map<String, String> replacements) { public static String replaceTokens(String input, Map<String, String> replacements) {
Matcher matcher = Patterns.TOKEN_PATTERN.matcher(input); Matcher matcher = Patterns.TOKEN_PATTERN.matcher(input);
StringBuilder builder = new StringBuilder(); StringBuilder builder = new StringBuilder();
int i = 0; int i = 0;
while (matcher.find()) { while (matcher.find()) {
String replacement = replacements.get(matcher.group(1)); String replacement = replacements.get(matcher.group(1));
builder.append(input.substring(i, matcher.start())); builder.append(input.substring(i, matcher.start()));
if (replacement == null) if (replacement == null)
builder.append(matcher.group(0)); builder.append(matcher.group(0));
else else
builder.append(replacement); builder.append(replacement);
i = matcher.end(); i = matcher.end();
} }
builder.append(input.substring(i, input.length())); builder.append(input.substring(i, input.length()));
return builder.toString(); return builder.toString();
} }
/**
* Will throw an exception if the argument is null or empty.
* @param nullableString
* string to verify. Can be null or empty.
*/
public static void checkNotEmpty(String nullableString) {
checkNotEmpty(nullableString, "Argument can't be null or empty");
}
/**
* Will throw an exception if the argument is null or empty. Accepts
* a custom error message.
* @param nullableString
* string to verify. Can be null or empty.
* @param message
* message to show in case of exception
*/
public static void checkNotEmpty(String nullableString, String message) {
checkArgument(nullableString != null && nullableString.length() > 0,
message);
}
} }

View File

@ -32,6 +32,8 @@ import org.jclouds.compute.domain.NodeMetadata;
import org.jclouds.compute.domain.NodeState; import org.jclouds.compute.domain.NodeState;
import org.jclouds.compute.domain.OsFamily; import org.jclouds.compute.domain.OsFamily;
import org.jclouds.compute.domain.Template; import org.jclouds.compute.domain.Template;
import org.jclouds.compute.options.RunScriptOptions;
import org.jclouds.domain.Credentials;
import org.jclouds.rest.RestContext; import org.jclouds.rest.RestContext;
import org.jclouds.ssh.jsch.config.JschSshClientModule; import org.jclouds.ssh.jsch.config.JschSshClientModule;
import org.testng.annotations.BeforeClass; import org.testng.annotations.BeforeClass;
@ -46,56 +48,61 @@ import com.google.common.collect.Iterables;
@Test(groups = "live", enabled = true, sequential = true, testName = "gogrid.GoGridComputeServiceLiveTest") @Test(groups = "live", enabled = true, sequential = true, testName = "gogrid.GoGridComputeServiceLiveTest")
public class GoGridComputeServiceLiveTest extends BaseComputeServiceLiveTest { public class GoGridComputeServiceLiveTest extends BaseComputeServiceLiveTest {
@BeforeClass @BeforeClass
@Override @Override
public void setServiceDefaults() { public void setServiceDefaults() {
service = "gogrid"; service = "gogrid";
} }
@Test @Test
public void testTemplateBuilder() { public void testTemplateBuilder() {
Template defaultTemplate = client.templateBuilder().build(); Template defaultTemplate = client.templateBuilder().build();
assertEquals(defaultTemplate.getImage().getArchitecture(), Architecture.X86_64); assertEquals(defaultTemplate.getImage().getArchitecture(), Architecture.X86_64);
assertEquals(defaultTemplate.getImage().getOsFamily(), OsFamily.CENTOS); assertEquals(defaultTemplate.getImage().getOsFamily(), OsFamily.CENTOS);
assertEquals(defaultTemplate.getLocation().getId(), "SANFRANCISCO"); assertEquals(defaultTemplate.getLocation().getId(), "SANFRANCISCO");
assertEquals(defaultTemplate.getSize().getCores(), 1.0d); assertEquals(defaultTemplate.getSize().getCores(), 1.0d);
} }
@Override @Override
protected JschSshClientModule getSshModule() { protected JschSshClientModule getSshModule() {
return new JschSshClientModule(); return new JschSshClientModule();
} }
public void testAssignability() throws Exception { public void testAssignability() throws Exception {
@SuppressWarnings("unused") @SuppressWarnings("unused")
RestContext<GoGridAsyncClient, GoGridClient> goGridContext = new ComputeServiceContextFactory() RestContext<GoGridAsyncClient, GoGridClient> goGridContext = new ComputeServiceContextFactory()
.createContext(service, user, password).getProviderSpecificContext(); .createContext(service, user, password).getProviderSpecificContext();
} }
@Test(enabled = true) @Test(enabled = true)
public void endToEndComputeServiceTest() { public void endToEndComputeServiceTest() {
ComputeService service = context.getComputeService(); ComputeService service = context.getComputeService();
Template t = service.templateBuilder().minRam(1024).imageId("1532").build(); Template t = service.templateBuilder().minRam(1024).imageId("1532").build();
assertEquals(t.getImage().getId(), "1532"); assertEquals(t.getImage().getId(), "1532");
service.runNodesWithTag(this.service, 1, t); service.runNodesWithTag(this.service, 3, t);
Map<String, ? extends ComputeMetadata> nodes = service.getNodes(); Map<String, ? extends ComputeMetadata> nodes = service.getNodes();
ComputeMetadata node = Iterables.find(nodes.values(), new Predicate<ComputeMetadata>() { ComputeMetadata node = Iterables.find(nodes.values(), new Predicate<ComputeMetadata>() {
@Override @Override
public boolean apply(ComputeMetadata computeMetadata) { public boolean apply(ComputeMetadata computeMetadata) {
return computeMetadata.getName().startsWith(GoGridComputeServiceLiveTest.this.service); return computeMetadata.getName().startsWith(GoGridComputeServiceLiveTest.this.service);
} }
}); });
NodeMetadata nodeMetadata = service.getNodeMetadata(node); NodeMetadata nodeMetadata = service.getNodeMetadata(node);
assertEquals(nodeMetadata.getPublicAddresses().size(), 1, assertEquals(nodeMetadata.getPublicAddresses().size(), 1,
"There must be 1 public address for the node"); "There must be 1 public address for the node");
assertTrue(nodeMetadata.getName().startsWith(this.service)); assertTrue(nodeMetadata.getName().startsWith(this.service));
service.rebootNode(nodeMetadata); // blocks until finished service.rebootNode(nodeMetadata); // blocks until finished
assertEquals(service.getNodeMetadata(nodeMetadata).getState(), NodeState.RUNNING); assertEquals(service.getNodeMetadata(nodeMetadata).getState(), NodeState.RUNNING);
service.destroyNode(nodeMetadata);
} client.runScriptOnNodesWithTag("gogrid", null/*no credentials*/,
"mkdir ~/ahha; sleep 3".getBytes(),
new RunScriptOptions.Builder().overrideCredentials(false).build());
service.destroyNodesWithTag("gogrid");
}
} }