Skip to content

Commit

Permalink
Fix word count
Browse files Browse the repository at this point in the history
Patch by  Ala' Alkhaldi, reviewed by brandonwilliams for CASSANDRA-7200
  • Loading branch information
driftx committed Jul 18, 2014
1 parent 062addb commit 337e4a8
Show file tree
Hide file tree
Showing 6 changed files with 38 additions and 8 deletions.
6 changes: 4 additions & 2 deletions examples/hadoop_cql3_word_count/README.txt
Original file line number Diff line number Diff line change
Expand Up @@ -36,8 +36,10 @@ Read the code in src/ for more details.
The word_count_counters example sums the counter columns for a row. The output
is written to a text file in /tmp/word_count_counters.

*If you want to point wordcount at a real cluster, modify the seed
and listenaddress settings accordingly.
*It is recommended to turn off vnodes when running Cassandra with hadoop.
This is done by setting "num_tokens: 1" in cassandra.yaml. If you want to
point wordcount at a real cluster, modify the seed and listenaddress
settings accordingly.


Troubleshooting
Expand Down
2 changes: 1 addition & 1 deletion examples/hadoop_cql3_word_count/conf/log4j.properties
Original file line number Diff line number Diff line change
Expand Up @@ -15,7 +15,7 @@
# specific language governing permissions and limitations
# under the License.

log4j.rootLogger=DEBUG,stdout,F
log4j.rootLogger=INFO,stdout,F

#stdout
log4j.appender.stdout=org.apache.log4j.ConsoleAppender
Expand Down
5 changes: 4 additions & 1 deletion examples/hadoop_cql3_word_count/src/WordCountSetup.java
Original file line number Diff line number Diff line change
Expand Up @@ -70,7 +70,10 @@ private static void setupKeyspace(Cassandra.Iface client)

client.execute_cql3_query(ByteBufferUtil.bytes(query), Compression.NONE, ConsistencyLevel.ONE);

int magnitude = client.describe_ring(WordCount.KEYSPACE).size();
String verifyQuery = "select count(*) from system.peers";
CqlResult result = client.execute_cql3_query(ByteBufferUtil.bytes(verifyQuery), Compression.NONE, ConsistencyLevel.ONE);

long magnitude = ByteBufferUtil.toLong(result.rows.get(0).columns.get(0).value);
try
{
Thread.sleep(1000 * magnitude);
Expand Down
6 changes: 4 additions & 2 deletions examples/hadoop_word_count/README.txt
Original file line number Diff line number Diff line change
Expand Up @@ -37,8 +37,10 @@ Read the code in src/ for more details.
The word_count_counters example sums the counter columns for a row. The output
is written to a text file in /tmp/word_count_counters.

*If you want to point wordcount at a real cluster, modify the seed
and listenaddress settings accordingly.
*It is recommended to turn off vnodes when running Cassandra with hadoop.
This is done by setting "num_tokens: 1" in cassandra.yaml. If you want to
point wordcount at a real cluster, modify the seed and listenaddress
settings accordingly.


Troubleshooting
Expand Down
2 changes: 1 addition & 1 deletion examples/hadoop_word_count/conf/log4j.properties
Original file line number Diff line number Diff line change
Expand Up @@ -15,7 +15,7 @@
# specific language governing permissions and limitations
# under the License.

log4j.rootLogger=DEBUG,stdout,F
log4j.rootLogger=INFO,stdout,F

#stdout
log4j.appender.stdout=org.apache.log4j.ConsoleAppender
Expand Down
25 changes: 24 additions & 1 deletion examples/hadoop_word_count/src/WordCountSetup.java
Original file line number Diff line number Diff line change
Expand Up @@ -176,10 +176,33 @@ private static void setupKeyspace(Cassandra.Iface client) throws TException, Inv
KsDef ksDef = new KsDef(WordCount.KEYSPACE, "org.apache.cassandra.locator.SimpleStrategy", cfDefList);
ksDef.putToStrategy_options("replication_factor", "1");
client.system_add_keyspace(ksDef);
int magnitude = client.describe_ring(WordCount.KEYSPACE).size();

int magnitude = getNumberOfHosts(client);
Uninterruptibles.sleepUninterruptibly(magnitude, TimeUnit.SECONDS);
}

private static int getNumberOfHosts(Cassandra.Iface client)
throws InvalidRequestException, UnavailableException, TimedOutException, TException
{
client.set_keyspace("system");
SlicePredicate predicate = new SlicePredicate();
SliceRange sliceRange = new SliceRange();
sliceRange.setStart(new byte[0]);
sliceRange.setFinish(new byte[0]);
predicate.setSlice_range(sliceRange);

KeyRange keyrRange = new KeyRange();
keyrRange.setStart_key(new byte[0]);
keyrRange.setEnd_key(new byte[0]);
//keyrRange.setCount(100);

ColumnParent parent = new ColumnParent("peers");

List<KeySlice> ls = client.get_range_slices(parent, predicate, keyrRange, ConsistencyLevel.ONE);

return ls.size();
}

private static Cassandra.Iface createConnection() throws TTransportException
{
if (System.getProperty("cassandra.host") == null || System.getProperty("cassandra.port") == null)
Expand Down

0 comments on commit 337e4a8

Please sign in to comment.