Skip to content

Commit

Permalink
wip, continue implementation of cluster resource
Browse files Browse the repository at this point in the history
  • Loading branch information
varjoranta committed Dec 2, 2014
1 parent 0aa5d70 commit cb02702
Show file tree
Hide file tree
Showing 13 changed files with 231 additions and 70 deletions.
46 changes: 26 additions & 20 deletions pom.xml
Original file line number Diff line number Diff line change
Expand Up @@ -31,6 +31,12 @@
<groupId>org.apache.cassandra</groupId>
<artifactId>cassandra-all</artifactId>
<version>${cassandra.version}</version>
<exclusions>
<exclusion>
<groupId>org.slf4j</groupId>
<artifactId>slf4j-log4j12</artifactId>
</exclusion>
</exclusions>
</dependency>
<dependency>
<groupId>junit</groupId>
Expand All @@ -41,26 +47,26 @@

<build>
<plugins>
<!--
<plugin>
<groupId>org.apache.maven.plugins</groupId>
<artifactId>maven-enforcer-plugin</artifactId>
<version>1.3.1</version>
<executions>
<execution>
<id>enforce</id>
<configuration>
<rules>
<DependencyConvergence/>
</rules>
</configuration>
<goals>
<goal>enforce</goal>
</goals>
</execution>
</executions>
</plugin>
-->
<!--
<plugin>
<groupId>org.apache.maven.plugins</groupId>
<artifactId>maven-enforcer-plugin</artifactId>
<version>1.3.1</version>
<executions>
<execution>
<id>enforce</id>
<configuration>
<rules>
<DependencyConvergence/>
</rules>
</configuration>
<goals>
<goal>enforce</goal>
</goals>
</execution>
</executions>
</plugin>
-->
<plugin>
<groupId>org.apache.maven.plugins</groupId>
<artifactId>maven-compiler-plugin</artifactId>
Expand Down
4 changes: 2 additions & 2 deletions src/main/java/com/spotify/reaper/ReaperApplication.java
Original file line number Diff line number Diff line change
@@ -1,6 +1,6 @@
package com.spotify.reaper;

import com.spotify.reaper.resources.AddClusterResource;
import com.spotify.reaper.resources.ClusterResource;
import com.spotify.reaper.resources.AddTableResource;
import com.spotify.reaper.resources.PingResource;
import com.spotify.reaper.resources.RepairTableResource;
Expand Down Expand Up @@ -40,7 +40,7 @@ public void run(ReaperApplicationConfiguration config,
IStorage storage = initializeStorage(config, environment);

final PingResource pingResource = new PingResource();
final AddClusterResource addClusterResource = new AddClusterResource(storage);
final ClusterResource addClusterResource = new ClusterResource(storage);
final AddTableResource addTableResource = new AddTableResource();
final RepairTableResource repairTableResource = new RepairTableResource();

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -26,6 +26,7 @@ public class ReaperApplicationConfiguration extends Configuration {

@Valid
@NotNull
@JsonProperty
private DataSourceFactory database = new DataSourceFactory();

@JsonProperty
Expand Down
17 changes: 15 additions & 2 deletions src/main/java/com/spotify/reaper/cassandra/ClusterInfo.java
Original file line number Diff line number Diff line change
Expand Up @@ -8,23 +8,36 @@ public class ClusterInfo implements IClusterInfo {

private String seedHost;
private int seedPort = 0;
private String clusterName;

private List<String> tokens;

public static ClusterInfo getInstance(String seedHost) throws ReaperException {
return new ClusterInfo(seedHost, 0).init();
}

public ClusterInfo(String seedHost, int seedPort) {
this.seedHost = seedHost;
this.seedPort = seedPort;
}

public void init() throws ReaperException {
JMXProxy jmx = seedPort == 0 ? JMXProxy.connect(seedHost) : JMXProxy.connect(seedHost, seedPort);
public ClusterInfo init() throws ReaperException {
JMXProxy jmx =
seedPort == 0 ? JMXProxy.connect(seedHost) : JMXProxy.connect(seedHost, seedPort);
tokens = jmx.getTokens();
clusterName = jmx.getClusterName();
jmx.close();
return this;
}

@Override
public List<String> getTokens() {
return tokens;
}

@Override
public String getClusterName() {
return clusterName;
}

}
Original file line number Diff line number Diff line change
Expand Up @@ -6,4 +6,5 @@ public interface IClusterInfo {

public List<String> getTokens();

public String getClusterName();
}
22 changes: 14 additions & 8 deletions src/main/java/com/spotify/reaper/cassandra/JMXProxy.java
Original file line number Diff line number Diff line change
@@ -1,7 +1,9 @@
package com.spotify.reaper.cassandra;

import com.google.common.collect.Lists;

import com.spotify.reaper.ReaperException;

import org.apache.cassandra.service.StorageServiceMBean;

import java.io.IOException;
Expand All @@ -20,11 +22,11 @@ public class JMXProxy {

private static final int DEFAULT_JMX_PORT = 7199;

private JMXConnector jmxc = null;
private JMXConnector jmxConnector = null;
private StorageServiceMBean ssProxy;

public JMXProxy(JMXConnector jmxc, StorageServiceMBean ssProxy) {
this.jmxc = jmxc;
public JMXProxy(JMXConnector jmxConnector, StorageServiceMBean ssProxy) {
this.jmxConnector = jmxConnector;
this.ssProxy = ssProxy;
}

Expand All @@ -37,17 +39,17 @@ public static JMXProxy connect(String host, int port) throws ReaperException {
ObjectName name;
try {
jmxUrl = new JMXServiceURL(String.format("service:jmx:rmi:///jndi/rmi://%s:%d/jmxrmi",
host, port));
host, port));
name = new ObjectName("org.apache.cassandra.db:type=StorageService");
} catch (MalformedURLException | MalformedObjectNameException e) {
throw new ReaperException("Failure during preparations for JMX connection", e);
}
try {
JMXConnector jmxc = JMXConnectorFactory.connect(jmxUrl);
MBeanServerConnection mbeanServerConn = jmxc.getMBeanServerConnection();
JMXConnector jmxConnector = JMXConnectorFactory.connect(jmxUrl);
MBeanServerConnection mbeanServerConn = jmxConnector.getMBeanServerConnection();
StorageServiceMBean
ssProxy = JMX.newMBeanProxy(mbeanServerConn, name, StorageServiceMBean.class);
return new JMXProxy(jmxc, ssProxy);
return new JMXProxy(jmxConnector, ssProxy);
} catch (IOException e) {
throw new ReaperException("Failure when establishing JMX connection", e);
}
Expand All @@ -57,9 +59,13 @@ public List<String> getTokens() {
return Lists.newArrayList(ssProxy.getTokenToEndpointMap().keySet());
}

public String getClusterName() {
return ssProxy.getClusterName();
}

public void close() throws ReaperException {
try {
jmxc.close();
jmxConnector.close();
} catch (IOException e) {
throw new ReaperException(e);
}
Expand Down
35 changes: 0 additions & 35 deletions src/main/java/com/spotify/reaper/resources/AddClusterResource.java

This file was deleted.

75 changes: 75 additions & 0 deletions src/main/java/com/spotify/reaper/resources/ClusterResource.java
Original file line number Diff line number Diff line change
@@ -0,0 +1,75 @@
package com.spotify.reaper.resources;

import com.google.common.base.Optional;

import com.spotify.reaper.ReaperException;
import com.spotify.reaper.cassandra.ClusterInfo;
import com.spotify.reaper.core.Cluster;
import com.spotify.reaper.storage.IStorage;

import org.slf4j.Logger;
import org.slf4j.LoggerFactory;

import java.net.URI;
import java.net.URL;

import javax.ws.rs.GET;
import javax.ws.rs.POST;
import javax.ws.rs.Path;
import javax.ws.rs.PathParam;
import javax.ws.rs.Produces;
import javax.ws.rs.QueryParam;
import javax.ws.rs.core.Context;
import javax.ws.rs.core.MediaType;
import javax.ws.rs.core.Response;
import javax.ws.rs.core.UriInfo;

@Path("/cluster/{name}")
@Produces(MediaType.APPLICATION_JSON)
public class ClusterResource {

private static final Logger LOG = LoggerFactory.getLogger(ClusterResource.class);

private final IStorage storage;

public ClusterResource(IStorage storage) {
this.storage = storage;
}

@GET
public Response getCluster(@PathParam("name") String name) {
Cluster cluster = storage.getCluster(name);
return Response.ok().entity(cluster).build();
}

@POST
public Response addCluster(@Context UriInfo uriInfo, @QueryParam("host") Optional<String> host) {
if (!host.isPresent()) {
LOG.error("POST on cluster resource called without host");
return Response.status(400).entity("query parameter \"host\" required").build();
}
LOG.info("add cluster called with host: {}", host);

ClusterInfo clusterInfo;
try {
clusterInfo = ClusterInfo.getInstance(host.get());
} catch (ReaperException e) {
String errMsg = "failed to get cluster info from seed host: " + host.get();
LOG.error(errMsg);
e.printStackTrace();
return Response.status(400).entity(errMsg).build();
}

URI createdURI = null;
try {
createdURI = (new URL(uriInfo.getAbsolutePath().toURL(), "1")).toURI();
} catch (Exception e) {
LOG.error("failed creating target URI: " + uriInfo.getAbsolutePath());
e.printStackTrace();
}

String replyMsg = "cluster with name \"" + clusterInfo.getClusterName() + "\" created";
return Response.created(createdURI).entity(replyMsg).build();
}

}
2 changes: 2 additions & 0 deletions src/main/java/com/spotify/reaper/storage/IStorage.java
Original file line number Diff line number Diff line change
Expand Up @@ -10,6 +10,8 @@ public interface IStorage {

public Cluster getCluster(String clusterName);

public Cluster insertCluster(Cluster newCluster);

public RepairRun addRepairRun(RepairRun repairRun);

public RepairRun getRepairRun(long id);
Expand Down
5 changes: 5 additions & 0 deletions src/main/java/com/spotify/reaper/storage/MemoryStorage.java
Original file line number Diff line number Diff line change
Expand Up @@ -13,6 +13,11 @@ public Cluster getCluster(String clusterName) {
return null;
}

@Override
public Cluster insertCluster(Cluster newCluster) {
return null;
}

@Override
public RepairRun addRepairRun(RepairRun repairRun) {
return null;
Expand Down
28 changes: 25 additions & 3 deletions src/main/java/com/spotify/reaper/storage/PostgresStorage.java
Original file line number Diff line number Diff line change
Expand Up @@ -4,8 +4,10 @@
import com.spotify.reaper.ReaperException;
import com.spotify.reaper.core.Cluster;
import com.spotify.reaper.core.RepairRun;
import com.spotify.reaper.storage.postgresql.IStoragePostgreSQL;

import org.skife.jdbi.v2.DBI;
import org.skife.jdbi.v2.Handle;
import org.slf4j.Logger;
import org.slf4j.LoggerFactory;

Expand All @@ -26,16 +28,36 @@ public PostgresStorage(ReaperApplicationConfiguration config, Environment enviro
try {
final DBIFactory factory = new DBIFactory();
jdbi = factory.build(environment, config.getDataSourceFactory(), "postgresql");
}
catch (ClassNotFoundException ex) {
} catch (ClassNotFoundException ex) {
LOG.error("failed creating database connection: {}", ex);
throw new ReaperException(ex);
}
}

@Override
public Cluster getCluster(String clusterName) {
return null;
Handle h = jdbi.open();
IStoragePostgreSQL postgres = h.attach(IStoragePostgreSQL.class);
Cluster result = postgres.getCluster(clusterName);
h.close();
return result;
}

@Override
public Cluster insertCluster(Cluster newCluster) {
Handle h = jdbi.open();
IStoragePostgreSQL postgres = h.attach(IStoragePostgreSQL.class);
int rowsAdded = postgres.insertCluster(newCluster);
Cluster result;
if (rowsAdded < 1) {
LOG.warn("failed inserting cluster with name: {}", newCluster.getName());
result = null;
}
else {
result = postgres.getCluster(newCluster.getName());
}
h.close();
return result;
}

@Override
Expand Down
Loading

0 comments on commit cb02702

Please sign in to comment.