working integration tests

This commit is contained in:
nishantmonu51 2014-11-06 23:53:24 +05:30 committed by Xavier Léauté
parent 5e7f68058d
commit f69e0591b0
52 changed files with 7925 additions and 0 deletions

View File

@ -0,0 +1,90 @@
Integration Testing
=========================
## Installing Docker and Running
Please refer to instructions at [https://github.com/druid-io/docker-druid/blob/master/docker-install.md](https://github.com/druid-io/docker-druid/blob/master/docker-install.md)
Instead of running
```
boot2docker init
```
run instead
```
boot2docker init -m 6000
```
Make sure that you have at least 6GB of memory available before you run the tests.
Set the docker ip via:
```
export DOCKER_IP=$(boot2docker ip 2>/dev/null)
```
Verify that docker is running by issuing the following command:
```
docker info
```
Running Integration tests
=========================
## Running tests using mvn
To run all the tests using mvn run the following command -
'''''
mvn verify -P integration-tests
'''''
To run only a single test using mvn run following command -
'''''
mvn verify -P integration-tests -Dit.test=<test_name>
'''''
Writing a New Test
===============
## What should we cover in integration tests
For every end-user functionality provided by druid we should have an integration-test verifying the correctness.
## Rules to be followed while writing a new integration test
### Every Integration Test must follow these rules
1) Name of the test must start with a prefix "IT"
2) A test should be independent of other tests
3) Tests are to be written in TestNG style ([http://testng.org/doc/documentation-main.html#methods](http://testng.org/doc/documentation-main.html#methods))
4) If a test loads some data it is the responsibility of the test to clean up the data from the cluster
### How to use Guice Dependency Injection in a test
A test can access different helper and utility classes provided by test-framework in order to access Coordinator,Broker etc..
To mark a test be able to use Guice Dependency Injection -
Annotate the test class with the below annotation
'''''''
@Guice(moduleFactory = DruidTestModuleFactory.class)
'''''''
This will tell the test framework that the test class needs to be constructed using guice.
### Helper Classes provided
1) IntegrationTestingConfig - configuration of the test
2) CoordinatorResourceTestClient - httpclient for coordinator endpoints
3) OverlordResourceTestClient - httpclient for indexer endpoints
4) QueryResourceTestClient - httpclient for broker endpoints
### Static Utility classes
1) RetryUtil - provides methods to retry an operation until it succeeds for configurable no. of times
2) FromFileTestQueryHelper - reads queries with expected results from file and executes them and verifies the results using ResultVerifier
Refer ITIndexerTest as an example on how to use dependency Injection
TODOS
=======================
1) Remove the patch for TestNG after resolution of Surefire-622

View File

@ -0,0 +1,71 @@
FROM ubuntu:14.04
# Add Java 7 repository
RUN apt-get update
RUN apt-get install -y software-properties-common
RUN apt-add-repository -y ppa:webupd8team/java
RUN apt-get update
# Oracle Java 7
RUN echo oracle-java-7-installer shared/accepted-oracle-license-v1-1 select true | /usr/bin/debconf-set-selections
RUN apt-get install -y oracle-java7-installer
RUN apt-get install -y oracle-java7-set-default
# MySQL (Metadata store)
RUN apt-get install -y mysql-server
# Supervisor
RUN apt-get install -y supervisor
# Maven
RUN wget -q -O - http://www.us.apache.org/dist/maven/maven-3/3.2.5/binaries/apache-maven-3.2.5-bin.tar.gz | tar -xzf - -C /usr/local
RUN ln -s /usr/local/apache-maven-3.2.1 /usr/local/apache-maven
RUN ln -s /usr/local/apache-maven/bin/mvn /usr/local/bin/mvn
# Zookeeper
RUN wget -q -O - http://www.us.apache.org/dist/zookeeper/zookeeper-3.4.6/zookeeper-3.4.6.tar.gz | tar -xzf - -C /usr/local
RUN cp /usr/local/zookeeper-3.4.6/conf/zoo_sample.cfg /usr/local/zookeeper-3.4.6/conf/zoo.cfg
RUN ln -s /usr/local/zookeeper-3.4.6 /usr/local/zookeeper
# git
RUN apt-get install -y git
# Druid system user
RUN adduser --system --group --no-create-home druid
RUN mkdir -p /var/lib/druid
RUN chown druid:druid /var/lib/druid
# Add druid jars
ADD lib/* /usr/local/druid/lib/
WORKDIR /
# Setup metadata store
RUN /etc/init.d/mysql start && echo "GRANT ALL ON druid.* TO 'druid'@'%' IDENTIFIED BY 'diurd'; CREATE database druid;" | mysql -u root && /etc/init.d/mysql stop
# Add sample data
RUN /etc/init.d/mysql start && java -Ddruid.metadata.storage.type=mysql -cp "/usr/local/druid/lib/*" io.druid.cli.Main tools metadata-init --connectURI="jdbc:mysql://localhost:3306/druid" --user=druid --password=diurd && /etc/init.d/mysql stop
ADD sample-data.sql sample-data.sql
RUN /etc/init.d/mysql start && cat sample-data.sql | mysql -u root druid && /etc/init.d/mysql stop
# Setup supervisord
ADD supervisord.conf /etc/supervisor/conf.d/supervisord.conf
# Clean up
RUN apt-get clean && rm -rf /tmp/* /var/tmp/*
# Expose ports:
# - 8081: HTTP (coordinator)
# - 8082: HTTP (broker)
# - 8083: HTTP (historical)
# - 3306: MySQL
# - 2181 2888 3888: ZooKeeper
# - 8100 8101 8102 8103 8104 : peon ports
EXPOSE 8081
EXPOSE 8082
EXPOSE 8083
EXPOSE 3306
EXPOSE 2181 2888 3888
EXPOSE 8100 8101 8102 8103 8104
WORKDIR /var/lib/druid
ENTRYPOINT export HOST_IP="$(resolveip -s $HOSTNAME)" && exec /usr/bin/supervisord -c /etc/supervisor/conf.d/supervisord.conf

View File

@ -0,0 +1,29 @@
[program:druid-broker]
command=java
-server
-Xmx1g
-Xms1g
-XX:NewSize=500m
-XX:MaxNewSize=500m
-XX:+UseConcMarkSweepGC
-XX:+PrintGCDetails
-XX:+PrintGCTimeStamps
-Duser.timezone=UTC
-Dfile.encoding=UTF-8
-Ddruid.host=%(ENV_HOST_IP)s
-Ddruid.zk.service.host=druid-zookeeper
-Ddruid.processing.buffer.sizeBytes=75000000
-Ddruid.server.http.numThreads=100
-Ddruid.processing.numThreads=1
-Ddruid.broker.http.numConnections=30
-Ddruid.broker.http.readTimeout=PT5M
-Ddruid.broker.cache.useCache=true
-Ddruid.broker.cache.populateCache=true
-Ddruid.cache.type=local
-Ddruid.cache.sizeInBytes=40000000
-cp /usr/local/druid/lib/*
io.druid.cli.Main server broker
redirect_stderr=true
autorestart=false
priority=100
stdout_logfile=/shared/logs/broker.log

View File

@ -0,0 +1,23 @@
[program:druid-coordinator]
command=java
-server
-Xmx128m
-Xms128m
-XX:+UseConcMarkSweepGC
-XX:+PrintGCDetails
-XX:+PrintGCTimeStamps
-Duser.timezone=UTC
-Dfile.encoding=UTF-8
-Ddruid.host=%(ENV_HOST_IP)s
-Ddruid.metadata.storage.type=mysql
-Ddruid.metadata.storage.connector.connectURI=jdbc:mysql://druid-metadata-storage/druid
-Ddruid.metadata.storage.connector.user=druid
-Ddruid.metadata.storage.connector.password=diurd
-Ddruid.zk.service.host=druid-zookeeper
-Ddruid.coordinator.startDelay=PT5S
-cp /usr/local/druid/lib/*
io.druid.cli.Main server coordinator
redirect_stderr=true
priority=100
autorestart=false
stdout_logfile=/shared/logs/coordinator.log

View File

@ -0,0 +1,27 @@
[program:druid-historical]
command=java
-server
-Xmx1500m
-Xms1500m
-XX:NewSize=750m
-XX:MaxNewSize=750m
-XX:+UseConcMarkSweepGC
-XX:+PrintGCDetails
-XX:+PrintGCTimeStamps
-Duser.timezone=UTC
-Dfile.encoding=UTF-8
-Ddruid.host=%(ENV_HOST_IP)s
-Ddruid.zk.service.host=druid-zookeeper
-Ddruid.s3.accessKey=AKIAIMKECRUYKDQGR6YQ
-Ddruid.s3.secretKey=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
-Ddruid.processing.buffer.sizeBytes=75000000
-Ddruid.processing.numThreads=3
-Ddruid.server.http.numThreads=100
-Ddruid.segmentCache.locations="[{\"path\":\"/shared/druid/indexCache\",\"maxSize\":5000000000}]"
-Ddruid.server.maxSize=5000000000
-cp /usr/local/druid/lib/*
io.druid.cli.Main server historical
redirect_stderr=true
priority=100
autorestart=false
stdout_logfile=/shared/logs/historical.log

View File

@ -0,0 +1,6 @@
[program:mysql]
command=/usr/bin/pidproxy /var/run/mysqld/mysqld.pid /usr/bin/mysqld_safe
--bind-address=0.0.0.0
user=mysql
priority=0
stdout_logfile=/shared/logs/mysql.log

View File

@ -0,0 +1,29 @@
[program:druid-middlemanager]
command=java
-server
-Xmx64m
-Xms64m
-XX:+UseConcMarkSweepGC
-XX:+PrintGCDetails
-XX:+PrintGCTimeStamps
-Duser.timezone=UTC
-Dfile.encoding=UTF-8
-Ddruid.host=%(ENV_HOST_IP)s
-Ddruid.zk.service.host=druid-zookeeper
-Ddruid.indexer.logs.directory=/shared/tasklogs
-Ddruid.storage.storageDirectory=/shared/storage
-Ddruid.indexer.runner.javaOpts=-server -Xmx256m -Xms256m -XX:NewSize=128m -XX:MaxNewSize=128m -XX:+UseConcMarkSweepGC -XX:+PrintGCDetails -XX:+PrintGCTimeStamps
-Ddruid.indexer.fork.property.druid.processing.buffer.sizeBytes=75000000
-Ddruid.indexer.fork.property.druid.processing.numThreads=1
-Ddruid.indexer.fork.server.http.numThreads=100
-Ddruid.s3.accessKey=AKIAIMKECRUYKDQGR6YQ
-Ddruid.s3.secretKey=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
-Ddruid.worker.ip=%(ENV_HOST_IP)s
-Ddruid.selectors.indexing.serviceName=druid:overlord
-Ddruid.indexer.task.chathandler.type=announce
-cp /usr/local/druid/lib/*
io.druid.cli.Main server middleManager
redirect_stderr=true
priority=100
autorestart=false
stdout_logfile=/shared/logs/middlemanager.log

View File

@ -0,0 +1,25 @@
[program:druid-overlord]
command=java
-server
-Xmx128m
-Xms128m
-XX:+UseConcMarkSweepGC
-XX:+PrintGCDetails
-XX:+PrintGCTimeStamps
-Duser.timezone=UTC
-Dfile.encoding=UTF-8
-Ddruid.host=%(ENV_HOST_IP)s
-Ddruid.metadata.storage.type=mysql
-Ddruid.metadata.storage.connector.connectURI=jdbc:mysql://druid-metadata-storage/druid
-Ddruid.metadata.storage.connector.user=druid
-Ddruid.metadata.storage.connector.password=diurd
-Ddruid.zk.service.host=druid-zookeeper
-Ddruid.indexer.storage.type=metadata
-Ddruid.indexer.logs.directory=/shared/tasklogs
-Ddruid.indexer.runner.type=remote
-cp /usr/local/druid/lib/*
io.druid.cli.Main server overlord
redirect_stderr=true
priority=100
autorestart=false
stdout_logfile=/shared/logs/overlord.log

View File

@ -0,0 +1,20 @@
[program:druid-router]
command=java
-server
-Xmx1g
-XX:+UseConcMarkSweepGC
-XX:+PrintGCDetails
-XX:+PrintGCTimeStamps
-Duser.timezone=UTC
-Dfile.encoding=UTF-8
-Ddruid.host=%(ENV_HOST_IP)s
-Ddruid.zk.service.host=druid-zookeeper
-Ddruid.computation.buffer.size=75000000
-Ddruid.server.http.numThreads=100
-Ddruid.processing.numThreads=1
-cp /usr/local/druid/lib/*
io.druid.cli.Main server router
redirect_stderr=true
priority=100
autorestart=false
stdout_logfile=/shared/logs/router.log

View File

@ -0,0 +1,5 @@
INSERT INTO druid_segments (id,dataSource,created_date,start,end,partitioned,version,used,payload) VALUES ('twitterstream_2013-01-01T00:00:00.000Z_2013-01-02T00:00:00.000Z_2013-01-02T04:13:41.980Z_v9','twitterstream','2013-05-13T01:08:18.192Z','2013-01-01T00:00:00.000Z','2013-01-02T00:00:00.000Z',0,'2013-01-02T04:13:41.980Z_v9',1,'{\"dataSource\":\"twitterstream\",\"interval\":\"2013-01-01T00:00:00.000Z/2013-01-02T00:00:00.000Z\",\"version\":\"2013-01-02T04:13:41.980Z_v9\",\"loadSpec\":{\"type\":\"s3_zip\",\"bucket\":\"static.druid.io\",\"key\":\"data/segments/twitterstream/2013-01-01T00:00:00.000Z_2013-01-02T00:00:00.000Z/2013-01-02T04:13:41.980Z_v9/0/index.zip\"},\"dimensions\":\"has_links,first_hashtag,user_time_zone,user_location,has_mention,user_lang,rt_name,user_name,is_retweet,is_viral,has_geo,url_domain,user_mention_name,reply_to_name\",\"metrics\":\"count,tweet_length,num_followers,num_links,num_mentions,num_hashtags,num_favorites,user_total_tweets\",\"shardSpec\":{\"type\":\"none\"},\"binaryVersion\":9,\"size\":445235220,\"identifier\":\"twitterstream_2013-01-01T00:00:00.000Z_2013-01-02T00:00:00.000Z_2013-01-02T04:13:41.980Z_v9\"}');
INSERT INTO druid_segments (id,dataSource,created_date,start,end,partitioned,version,used,payload) VALUES ('twitterstream_2013-01-02T00:00:00.000Z_2013-01-03T00:00:00.000Z_2013-01-03T03:44:58.791Z_v9','twitterstream','2013-05-13T00:03:28.640Z','2013-01-02T00:00:00.000Z','2013-01-03T00:00:00.000Z',0,'2013-01-03T03:44:58.791Z_v9',1,'{\"dataSource\":\"twitterstream\",\"interval\":\"2013-01-02T00:00:00.000Z/2013-01-03T00:00:00.000Z\",\"version\":\"2013-01-03T03:44:58.791Z_v9\",\"loadSpec\":{\"type\":\"s3_zip\",\"bucket\":\"static.druid.io\",\"key\":\"data/segments/twitterstream/2013-01-02T00:00:00.000Z_2013-01-03T00:00:00.000Z/2013-01-03T03:44:58.791Z_v9/0/index.zip\"},\"dimensions\":\"has_links,first_hashtag,user_time_zone,user_location,has_mention,user_lang,rt_name,user_name,is_retweet,is_viral,has_geo,url_domain,user_mention_name,reply_to_name\",\"metrics\":\"count,tweet_length,num_followers,num_links,num_mentions,num_hashtags,num_favorites,user_total_tweets\",\"shardSpec\":{\"type\":\"none\"},\"binaryVersion\":9,\"size\":435325540,\"identifier\":\"twitterstream_2013-01-02T00:00:00.000Z_2013-01-03T00:00:00.000Z_2013-01-03T03:44:58.791Z_v9\"}');
INSERT INTO druid_segments (id,dataSource,created_date,start,end,partitioned,version,used,payload) VALUES ('twitterstream_2013-01-03T00:00:00.000Z_2013-01-04T00:00:00.000Z_2013-01-04T04:09:13.590Z_v9','twitterstream','2013-05-13T00:03:48.807Z','2013-01-03T00:00:00.000Z','2013-01-04T00:00:00.000Z',0,'2013-01-04T04:09:13.590Z_v9',1,'{\"dataSource\":\"twitterstream\",\"interval\":\"2013-01-03T00:00:00.000Z/2013-01-04T00:00:00.000Z\",\"version\":\"2013-01-04T04:09:13.590Z_v9\",\"loadSpec\":{\"type\":\"s3_zip\",\"bucket\":\"static.druid.io\",\"key\":\"data/segments/twitterstream/2013-01-03T00:00:00.000Z_2013-01-04T00:00:00.000Z/2013-01-04T04:09:13.590Z_v9/0/index.zip\"},\"dimensions\":\"has_links,first_hashtag,user_time_zone,user_location,has_mention,user_lang,rt_name,user_name,is_retweet,is_viral,has_geo,url_domain,user_mention_name,reply_to_name\",\"metrics\":\"count,tweet_length,num_followers,num_links,num_mentions,num_hashtags,num_favorites,user_total_tweets\",\"shardSpec\":{\"type\":\"none\"},\"binaryVersion\":9,\"size\":411651320,\"identifier\":\"twitterstream_2013-01-03T00:00:00.000Z_2013-01-04T00:00:00.000Z_2013-01-04T04:09:13.590Z_v9\"}');
INSERT INTO druid_segments (id,dataSource,created_date,start,end,partitioned,version,used,payload) VALUES ('wikipedia_editstream_2012-12-29T00:00:00.000Z_2013-01-10T08:00:00.000Z_2013-01-10T08:13:47.830Z_v9','wikipedia_editstream','2013-03-15T20:49:52.348Z','2012-12-29T00:00:00.000Z','2013-01-10T08:00:00.000Z',0,'2013-01-10T08:13:47.830Z_v9',1,'{\"dataSource\":\"wikipedia_editstream\",\"interval\":\"2012-12-29T00:00:00.000Z/2013-01-10T08:00:00.000Z\",\"version\":\"2013-01-10T08:13:47.830Z_v9\",\"loadSpec\":{\"type\":\"s3_zip\",\"bucket\":\"static.druid.io\",\"key\":\"data/segments/wikipedia_editstream/2012-12-29T00:00:00.000Z_2013-01-10T08:00:00.000Z/2013-01-10T08:13:47.830Z_v9/0/index.zip\"},\"dimensions\":\"anonymous,area_code,city,continent_code,country_name,dma_code,geo,language,namespace,network,newpage,page,postal_code,region_lookup,robot,unpatrolled,user\",\"metrics\":\"added,count,deleted,delta,delta_hist,unique_users,variation\",\"shardSpec\":{\"type\":\"none\"},\"binaryVersion\":9,\"size\":446027801,\"identifier\":\"wikipedia_editstream_2012-12-29T00:00:00.000Z_2013-01-10T08:00:00.000Z_2013-01-10T08:13:47.830Z_v9\"}');
INSERT INTO druid_segments (id, dataSource, created_date, start, end, partitioned, version, used, payload) VALUES ('wikipedia_2013-08-01T00:00:00.000Z_2013-08-02T00:00:00.000Z_2013-08-08T21:22:48.989Z', 'wikipedia', '2013-08-08T21:26:23.799Z', '2013-08-01T00:00:00.000Z', '2013-08-02T00:00:00.000Z', '0', '2013-08-08T21:22:48.989Z', '1', '{\"dataSource\":\"wikipedia\",\"interval\":\"2013-08-01T00:00:00.000Z/2013-08-02T00:00:00.000Z\",\"version\":\"2013-08-08T21:22:48.989Z\",\"loadSpec\":{\"type\":\"s3_zip\",\"bucket\":\"static.druid.io\",\"key\":\"data/segments/wikipedia/20130801T000000.000Z_20130802T000000.000Z/2013-08-08T21_22_48.989Z/0/index.zip\"},\"dimensions\":\"dma_code,continent_code,geo,area_code,robot,country_name,network,city,namespace,anonymous,unpatrolled,page,postal_code,language,newpage,user,region_lookup\",\"metrics\":\"count,delta,variation,added,deleted\",\"shardSpec\":{\"type\":\"none\"},\"binaryVersion\":9,\"size\":24664730,\"identifier\":\"wikipedia_2013-08-01T00:00:00.000Z_2013-08-02T00:00:00.000Z_2013-08-08T21:22:48.989Z\"}');

View File

@ -0,0 +1,6 @@
[supervisord]
nodaemon=true
[include]
files = /usr/lib/druid/conf/*.conf

View File

@ -0,0 +1,5 @@
[program:zookeeper]
command=/usr/local/zookeeper/bin/zkServer.sh start-foreground
user=daemon
priority=0
stdout_logfile=/shared/logs/zookeeper.log

151
integration-tests/pom.xml Normal file
View File

@ -0,0 +1,151 @@
<?xml version="1.0" encoding="UTF-8"?>
<!--
~ Druid - a distributed column store.
~ Copyright (C) 2012, 2013 Metamarkets Group Inc.
~
~ This program is free software; you can redistribute it and/or
~ modify it under the terms of the GNU General Public License
~ as published by the Free Software Foundation; either version 2
~ of the License, or (at your option) any later version.
~
~ This program is distributed in the hope that it will be useful,
~ but WITHOUT ANY WARRANTY; without even the implied warranty of
~ MERCHANTABILITY or FITNESS FOR A PARTICULAR PURPOSE. See the
~ GNU General Public License for more details.
~
~ You should have received a copy of the GNU General Public License
~ along with this program; if not, write to the Free Software
~ Foundation, Inc., 51 Franklin Street, Fifth Floor, Boston, MA 02110-1301, USA.
-->
<project xmlns="http://maven.apache.org/POM/4.0.0" xmlns:xsi="http://www.w3.org/2001/XMLSchema-instance"
xsi:schemaLocation="http://maven.apache.org/POM/4.0.0 http://maven.apache.org/maven-v4_0_0.xsd">
<modelVersion>4.0.0</modelVersion>
<groupId>io.druid</groupId>
<artifactId>druid-integration-tests</artifactId>
<name>druid-integration-tests</name>
<description>druid-integration-tests</description>
<parent>
<groupId>io.druid</groupId>
<artifactId>druid</artifactId>
<version>0.7.0-SNAPSHOT</version>
</parent>
<dependencies>
<dependency>
<groupId>io.druid</groupId>
<artifactId>druid-common</artifactId>
<version>${project.parent.version}</version>
</dependency>
<dependency>
<groupId>io.druid.extensions</groupId>
<artifactId>druid-s3-extensions</artifactId>
<version>${project.parent.version}</version>
</dependency>
<dependency>
<groupId>io.druid.extensions</groupId>
<artifactId>druid-histogram</artifactId>
<version>${project.parent.version}</version>
</dependency>
<dependency>
<groupId>io.druid.extensions</groupId>
<artifactId>mysql-metadata-storage</artifactId>
<version>${project.parent.version}</version>
</dependency>
<dependency>
<groupId>io.druid</groupId>
<artifactId>druid-services</artifactId>
<version>${project.parent.version}</version>
</dependency>
<dependency>
<groupId>io.druid</groupId>
<artifactId>druid-server</artifactId>
<version>${project.parent.version}</version>
</dependency>
<!-- Tests -->
<dependency>
<groupId>org.testng</groupId>
<artifactId>testng</artifactId>
</dependency>
<dependency>
<groupId>org.easymock</groupId>
<artifactId>easymock</artifactId>
<scope>test</scope>
</dependency>
</dependencies>
<build>
<plugins>
<plugin>
<artifactId>maven-surefire-plugin</artifactId>
<configuration>
<excludes>
<exclude>**/IT*.java</exclude>
</excludes>
<systemPropertyVariables>
<user.timezone>UTC</user.timezone>
</systemPropertyVariables>
</configuration>
</plugin>
</plugins>
</build>
<profiles>
<profile>
<id>integration-tests</id>
<build>
<plugins>
<plugin>
<groupId>org.codehaus.mojo</groupId>
<artifactId>exec-maven-plugin</artifactId>
<version>1.2.1</version>
<executions>
<execution>
<id>build-and-start-druid-cluster</id>
<goals>
<goal>exec</goal>
</goals>
<phase>pre-integration-test</phase>
<configuration>
<executable>${project.basedir}/run_cluster.sh</executable>
</configuration>
</execution>
<execution>
<id>stop-druid-cluster</id>
<goals>
<goal>exec</goal>
</goals>
<phase>post-integration-test</phase>
<configuration>
<executable>${project.basedir}/stop_cluster.sh</executable>
</configuration>
</execution>
</executions>
</plugin>
<plugin>
<artifactId>maven-failsafe-plugin</artifactId>
<executions>
<execution>
<id>integration-tests</id>
<phase>integration-test</phase>
<goals>
<goal>integration-test</goal>
<goal>verify</goal>
</goals>
</execution>
</executions>
<configuration>
<argLine>-Duser.timezone=UTC -Dfile.encoding=UTF-8 -Dtestrunfactory=org.testng.DruidTestRunnerFactory
-Ddruid.test.config.dockerIp=${env.DOCKER_IP} -Ddruid.zk.service.host=${env.DOCKER_IP}
</argLine>
<suiteXmlFiles>
<suiteXmlFile>src/test/resources/testng.xml</suiteXmlFile>
</suiteXmlFiles>
</configuration>
</plugin>
</plugins>
</build>
</profile>
</profiles>
</project>

View File

@ -0,0 +1,49 @@
# cleanup
for node in druid-historical druid-coordinator druid-overlord druid-router druid-broker druid-middlemanager druid-zookeeper druid-metadata-storage;
do
docker stop $node
docker rm $node
done
# environment variables
DIR=$(cd $(dirname $0) && pwd)
DOCKERDIR=$DIR/docker
SHARED_DIR=${HOME}/shared
SUPERVISORDIR=/usr/lib/druid/conf
RESOURCEDIR=$DIR/src/test/resources
# Make directories if they dont exist
mkdir -p $SHARED_DIR/logs
mkdir -p $SHARED_DIR/tasklogs
# install druid jars
rm -rf $SHARED_DIR/docker
cp -R docker $SHARED_DIR/docker
mvn dependency:copy-dependencies -DoutputDirectory=$SHARED_DIR/docker/lib
# Build Druid Cluster Image
docker build -t druid/cluster $SHARED_DIR/docker
# Start zookeeper
docker run -d --name druid-zookeeper -p 2181:2181 -v $SHARED_DIR:/shared -v $DOCKERDIR/zookeeper.conf:$SUPERVISORDIR/zookeeper.conf druid/cluster
# Start MYSQL
docker run -d --name druid-metadata-storage -v $SHARED_DIR:/shared -v $DOCKERDIR/metadata-storage.conf:$SUPERVISORDIR/metadata-storage.conf druid/cluster
# Start Overlord
docker run -d --name druid-overlord -p 8090:8090 -v $SHARED_DIR:/shared -v $DOCKERDIR/overlord.conf:$SUPERVISORDIR/overlord.conf --link druid-metadata-storage:druid-metadata-storage --link druid-zookeeper:druid-zookeeper druid/cluster
# Start Coordinator
docker run -d --name druid-coordinator -p 8081:8081 -v $SHARED_DIR:/shared -v $DOCKERDIR/coordinator.conf:$SUPERVISORDIR/coordinator.conf --link druid-overlord:druid-overlord --link druid-metadata-storage:druid-metadata-storage --link druid-zookeeper:druid-zookeeper druid/cluster
# Start Historical
docker run -d --name druid-historical -v $SHARED_DIR:/shared -v $DOCKERDIR/historical.conf:$SUPERVISORDIR/historical.conf --link druid-zookeeper:druid-zookeeper druid/cluster
# Start Middlemanger
docker run -d --name druid-middlemanager -p 8100:8100 -p 8101:8101 -p 8102:8102 -p 8103:8103 -p 8104:8104 -p 8105:8105 -v $RESOURCEDIR:/resources -v $SHARED_DIR:/shared -v $DOCKERDIR/middlemanager.conf:$SUPERVISORDIR/middlemanager.conf --link druid-zookeeper:druid-zookeeper --link druid-overlord:druid-overlord druid/cluster
# Start Broker
docker run -d --name druid-broker -v $SHARED_DIR:/shared -v $DOCKERDIR/broker.conf:$SUPERVISORDIR/broker.conf --link druid-zookeeper:druid-zookeeper --link druid-middlemanager:druid-middlemanager --link druid-historical:druid-historical druid/cluster
# Start Router
docker run -d --name druid-router -p 8888:8888 -v $SHARED_DIR:/shared -v $DOCKERDIR/router.conf:$SUPERVISORDIR/router.conf --link druid-zookeeper:druid-zookeeper --link druid-coordinator:druid-coordinator --link druid-broker:druid-broker druid/cluster

View File

@ -0,0 +1,67 @@
/*
* Druid - a distributed column store.
* Copyright (C) 2012, 2013 Metamarkets Group Inc.
*
* This program is free software; you can redistribute it and/or
* modify it under the terms of the GNU General Public License
* as published by the Free Software Foundation; either version 2
* of the License, or (at your option) any later version.
*
* This program is distributed in the hope that it will be useful,
* but WITHOUT ANY WARRANTY; without even the implied warranty of
* MERCHANTABILITY or FITNESS FOR A PARTICULAR PURPOSE. See the
* GNU General Public License for more details.
*
* You should have received a copy of the GNU General Public License
* along with this program; if not, write to the Free Software
* Foundation, Inc., 51 Franklin Street, Fifth Floor, Boston, MA 02110-1301, USA.
*/
package io.druid.testing;
import com.fasterxml.jackson.annotation.JsonProperty;
import com.google.api.client.repackaged.com.google.common.base.Throwables;
import org.apache.commons.io.IOUtils;
import javax.validation.constraints.NotNull;
import java.util.List;
public class DockerConfigProvider implements IntegrationTestingConfigProvider
{
@JsonProperty
@NotNull
private String dockerIp;
@Override
public IntegrationTestingConfig get()
{
return new IntegrationTestingConfig()
{
@Override
public String getCoordinatorHost()
{
return dockerIp+":8081";
}
@Override
public String getIndexerHost()
{
return dockerIp+":8090";
}
@Override
public String getRouterHost()
{
return dockerIp+ ":8888";
}
@Override
public String getMiddleManagerHost()
{
return dockerIp;
}
};
}
}

View File

@ -0,0 +1,33 @@
/*
* Druid - a distributed column store.
* Copyright (C) 2012, 2013 Metamarkets Group Inc.
*
* This program is free software; you can redistribute it and/or
* modify it under the terms of the GNU General Public License
* as published by the Free Software Foundation; either version 2
* of the License, or (at your option) any later version.
*
* This program is distributed in the hope that it will be useful,
* but WITHOUT ANY WARRANTY; without even the implied warranty of
* MERCHANTABILITY or FITNESS FOR A PARTICULAR PURPOSE. See the
* GNU General Public License for more details.
*
* You should have received a copy of the GNU General Public License
* along with this program; if not, write to the Free Software
* Foundation, Inc., 51 Franklin Street, Fifth Floor, Boston, MA 02110-1301, USA.
*/
package io.druid.testing;
/**
*/
public interface IntegrationTestingConfig
{
public String getCoordinatorHost();
public String getIndexerHost();
public String getRouterHost();
public String getMiddleManagerHost();
}

View File

@ -0,0 +1,32 @@
/*
* Druid - a distributed column store.
* Copyright (C) 2012, 2013 Metamarkets Group Inc.
*
* This program is free software; you can redistribute it and/or
* modify it under the terms of the GNU General Public License
* as published by the Free Software Foundation; either version 2
* of the License, or (at your option) any later version.
*
* This program is distributed in the hope that it will be useful,
* but WITHOUT ANY WARRANTY; without even the implied warranty of
* MERCHANTABILITY or FITNESS FOR A PARTICULAR PURPOSE. See the
* GNU General Public License for more details.
*
* You should have received a copy of the GNU General Public License
* along with this program; if not, write to the Free Software
* Foundation, Inc., 51 Franklin Street, Fifth Floor, Boston, MA 02110-1301, USA.
*/
package io.druid.testing;
import com.fasterxml.jackson.annotation.JsonSubTypes;
import com.fasterxml.jackson.annotation.JsonTypeInfo;
import com.google.inject.Provider;
@JsonTypeInfo(use = JsonTypeInfo.Id.NAME, property = "type", defaultImpl = DockerConfigProvider.class)
@JsonSubTypes(value = {
@JsonSubTypes.Type(name = "docker", value = DockerConfigProvider.class)
})
public interface IntegrationTestingConfigProvider extends Provider<IntegrationTestingConfig>
{
}

View File

@ -0,0 +1,146 @@
/*
* Druid - a distributed column store.
* Copyright (C) 2012, 2013 Metamarkets Group Inc.
*
* This program is free software; you can redistribute it and/or
* modify it under the terms of the GNU General Public License
* as published by the Free Software Foundation; either version 2
* of the License, or (at your option) any later version.
*
* This program is distributed in the hope that it will be useful,
* but WITHOUT ANY WARRANTY; without even the implied warranty of
* MERCHANTABILITY or FITNESS FOR A PARTICULAR PURPOSE. See the
* GNU General Public License for more details.
*
* You should have received a copy of the GNU General Public License
* along with this program; if not, write to the Free Software
* Foundation, Inc., 51 Franklin Street, Fifth Floor, Boston, MA 02110-1301, USA.
*/
package io.druid.testing.clients;
import com.fasterxml.jackson.core.type.TypeReference;
import com.fasterxml.jackson.databind.ObjectMapper;
import com.google.common.base.Charsets;
import com.google.common.base.Throwables;
import com.google.inject.Inject;
import com.metamx.common.ISE;
import com.metamx.common.logger.Logger;
import com.metamx.http.client.HttpClient;
import com.metamx.http.client.RequestBuilder;
import com.metamx.http.client.response.StatusResponseHandler;
import com.metamx.http.client.response.StatusResponseHolder;
import io.druid.guice.annotations.Global;
import io.druid.testing.IntegrationTestingConfig;
import org.jboss.netty.handler.codec.http.HttpMethod;
import org.jboss.netty.handler.codec.http.HttpResponseStatus;
import org.joda.time.Interval;
import java.net.URL;
import java.net.URLEncoder;
import java.util.Map;
public class CoordinatorResourceTestClient
{
private final static Logger LOG = new Logger(CoordinatorResourceTestClient.class);
private final ObjectMapper jsonMapper;
private final HttpClient httpClient;
private final String coordinator;
private final StatusResponseHandler responseHandler;
@Inject
CoordinatorResourceTestClient(
ObjectMapper jsonMapper,
@Global HttpClient httpClient, IntegrationTestingConfig config
)
{
this.jsonMapper = jsonMapper;
this.httpClient = httpClient;
this.coordinator = config.getCoordinatorHost();
this.responseHandler = new StatusResponseHandler(Charsets.UTF_8);
}
private String getCoordinatorURL()
{
return String.format(
"http://%s/druid/coordinator/v1/",
coordinator
);
}
private Map<String, Integer> getLoadStatus()
{
Map<String, Integer> status = null;
try {
StatusResponseHolder response = makeRequest(HttpMethod.GET, getCoordinatorURL() + "loadstatus?simple");
status = jsonMapper.readValue(
response.getContent(), new TypeReference<Map<String, Integer>>()
{
}
);
}
catch (Exception e) {
Throwables.propagate(e);
}
return status;
}
public boolean areSegmentsLoaded(String dataSource)
{
final Map<String, Integer> status = getLoadStatus();
return (status.containsKey(dataSource) && status.get(dataSource) == 0);
}
public void unloadSegmentsForDataSource(String dataSource, Interval interval)
{
killDataSource(dataSource, false, interval);
}
public void deleteSegmentsDataSource(String dataSource, Interval interval)
{
killDataSource(dataSource, true, interval);
}
private void killDataSource(String dataSource, boolean kill, Interval interval)
{
try {
makeRequest(
HttpMethod.DELETE,
String.format(
"%sdatasources/%s?kill=%s&interval=%s",
getCoordinatorURL(),
dataSource, kill, URLEncoder.encode(interval.toString(), "UTF-8")
)
);
}
catch (Exception e) {
throw Throwables.propagate(e);
}
}
private StatusResponseHolder makeRequest(HttpMethod method, String url)
{
try {
StatusResponseHolder response = new RequestBuilder(
this.httpClient,
method, new URL(url)
)
.go(responseHandler)
.get();
if (!response.getStatus().equals(HttpResponseStatus.OK)) {
throw new ISE(
"Error while making request to url[%s] status[%s] content[%s]",
url,
response.getStatus(),
response.getContent()
);
}
return response;
}
catch (Exception e) {
LOG.error(e, "Exception while sending request");
throw Throwables.propagate(e);
}
}
}

View File

@ -0,0 +1,135 @@
/*
* Druid - a distributed column store.
* Copyright (C) 2012, 2013 Metamarkets Group Inc.
*
* This program is free software; you can redistribute it and/or
* modify it under the terms of the GNU General Public License
* as published by the Free Software Foundation; either version 2
* of the License, or (at your option) any later version.
*
* This program is distributed in the hope that it will be useful,
* but WITHOUT ANY WARRANTY; without even the implied warranty of
* MERCHANTABILITY or FITNESS FOR A PARTICULAR PURPOSE. See the
* GNU General Public License for more details.
*
* You should have received a copy of the GNU General Public License
* along with this program; if not, write to the Free Software
* Foundation, Inc., 51 Franklin Street, Fifth Floor, Boston, MA 02110-1301, USA.
*/
package io.druid.testing.clients;
import com.fasterxml.jackson.core.type.TypeReference;
import com.fasterxml.jackson.databind.ObjectMapper;
import com.google.api.client.util.Charsets;
import com.google.common.base.Throwables;
import com.metamx.common.ISE;
import com.metamx.http.client.HttpClient;
import com.metamx.http.client.response.StatusResponseHandler;
import com.metamx.http.client.response.StatusResponseHolder;
import org.jboss.netty.handler.codec.http.HttpResponseStatus;
import javax.ws.rs.core.MediaType;
import java.io.BufferedReader;
import java.io.InputStreamReader;
import java.net.URL;
import java.util.ArrayList;
import java.util.Collection;
import java.util.Map;
public class EventReceiverFirehoseTestClient
{
private final String host;
private final StatusResponseHandler responseHandler;
private final ObjectMapper jsonMapper;
private final HttpClient httpClient;
private final String chatID;
public EventReceiverFirehoseTestClient(String host, String chatID, ObjectMapper jsonMapper, HttpClient httpClient)
{
this.host = host;
this.jsonMapper = jsonMapper;
this.responseHandler = new StatusResponseHandler(Charsets.UTF_8);
this.httpClient = httpClient;
this.chatID = chatID;
}
private String getURL()
{
return String.format(
"http://%s/druid/worker/v1/chat/%s/push-events/",
host,
chatID
);
}
/**
* post events from the collection and return the count of events accepted
*
* @param events Collection of events to be posted
*
* @return
*/
public int postEvents(Collection<Map<String, Object>> events)
{
try {
StatusResponseHolder response = httpClient.post(new URL(getURL()))
.setContent(
MediaType.APPLICATION_JSON,
this.jsonMapper.writeValueAsBytes(events)
)
.go(responseHandler)
.get();
if (!response.getStatus().equals(HttpResponseStatus.OK)) {
throw new ISE(
"Error while posting events to url[%s] status[%s] content[%s]",
getURL(),
response.getStatus(),
response.getContent()
);
}
Map<String, Integer> responseData = jsonMapper.readValue(
response.getContent(), new TypeReference<Map<String, Integer>>()
{
}
);
return responseData.get("eventCount");
}
catch (Exception e) {
throw Throwables.propagate(e);
}
}
public int postEventsFromFile(String file)
{
try {
BufferedReader reader = new BufferedReader(
new InputStreamReader(
EventReceiverFirehoseTestClient.class.getResourceAsStream(
file
)
)
);
String s;
Collection<Map<String, Object>> events = new ArrayList<Map<String, Object>>();
while ((s = reader.readLine()) != null) {
events.add(
(Map<String, Object>) this.jsonMapper.readValue(
s, new TypeReference<Map<String, Object>>()
{
}
)
);
}
int eventsPosted = postEvents(events);
if (eventsPosted != events.size()) {
throw new ISE("All events not posted, expected : %d actual : %d", events.size(), eventsPosted);
}
return eventsPosted;
}
catch (Exception e) {
throw Throwables.propagate(e);
}
}
}

View File

@ -0,0 +1,213 @@
/*
* Druid - a distributed column store.
* Copyright (C) 2012, 2013 Metamarkets Group Inc.
*
* This program is free software; you can redistribute it and/or
* modify it under the terms of the GNU General Public License
* as published by the Free Software Foundation; either version 2
* of the License, or (at your option) any later version.
*
* This program is distributed in the hope that it will be useful,
* but WITHOUT ANY WARRANTY; without even the implied warranty of
* MERCHANTABILITY or FITNESS FOR A PARTICULAR PURPOSE. See the
* GNU General Public License for more details.
*
* You should have received a copy of the GNU General Public License
* along with this program; if not, write to the Free Software
* Foundation, Inc., 51 Franklin Street, Fifth Floor, Boston, MA 02110-1301, USA.
*/
package io.druid.testing.clients;
import com.fasterxml.jackson.core.type.TypeReference;
import com.fasterxml.jackson.databind.ObjectMapper;
import com.google.common.base.Charsets;
import com.google.common.base.Throwables;
import com.google.inject.Inject;
import com.metamx.common.ISE;
import com.metamx.common.logger.Logger;
import com.metamx.http.client.HttpClient;
import com.metamx.http.client.response.StatusResponseHandler;
import com.metamx.http.client.response.StatusResponseHolder;
import io.druid.guice.annotations.Global;
import io.druid.indexing.common.TaskStatus;
import io.druid.indexing.common.task.Task;
import io.druid.testing.IntegrationTestingConfig;
import io.druid.testing.utils.RetryUtil;
import org.jboss.netty.handler.codec.http.HttpResponseStatus;
import java.net.URL;
import java.net.URLEncoder;
import java.util.List;
import java.util.Map;
import java.util.concurrent.Callable;
public class OverlordResourceTestClient
{
private final static Logger LOG = new Logger(OverlordResourceTestClient.class);
private final ObjectMapper jsonMapper;
private final HttpClient httpClient;
private final String indexer;
private final StatusResponseHandler responseHandler;
@Inject
OverlordResourceTestClient(
ObjectMapper jsonMapper,
@Global HttpClient httpClient, IntegrationTestingConfig config
)
{
this.jsonMapper = jsonMapper;
this.httpClient = httpClient;
this.indexer = config.getIndexerHost();
this.responseHandler = new StatusResponseHandler(Charsets.UTF_8);
}
private String getIndexerURL()
{
return String.format(
"http://%s/druid/indexer/v1/",
indexer
);
}
public String submitTask(Task task)
{
try {
return submitTask(this.jsonMapper.writeValueAsString(task));
}
catch (Exception e) {
throw Throwables.propagate(e);
}
}
public String submitTask(String task)
{
try {
StatusResponseHolder response = httpClient.post(new URL(getIndexerURL() + "task"))
.setContent(
"application/json",
task.getBytes()
)
.go(responseHandler)
.get();
if (!response.getStatus().equals(HttpResponseStatus.OK)) {
throw new ISE(
"Error while submitting task to indexer response [%s %s]",
response.getStatus(),
response.getContent()
);
}
Map<String, String> responseData = jsonMapper.readValue(
response.getContent(), new TypeReference<Map<String, String>>()
{
}
);
String taskID = responseData.get("task");
LOG.info("Submitted task with TaskID[%s]", taskID);
return taskID;
}
catch (Exception e) {
throw Throwables.propagate(e);
}
}
public TaskStatus.Status getTaskStatus(String taskID)
{
try {
StatusResponseHolder response = makeRequest(
String.format(
"%stask/%s/status",
getIndexerURL(),
URLEncoder.encode(taskID, "UTF-8")
)
);
LOG.info("Index status response" + response.getContent());
Map<String, Object> responseData = jsonMapper.readValue(
response.getContent(), new TypeReference<Map<String, Object>>()
{
}
);
//TODO: figure out a better way to parse the response...
String status = (String) ((Map) responseData.get("status")).get("status");
return TaskStatus.Status.valueOf(status);
}
catch (Exception e) {
throw Throwables.propagate(e);
}
}
public List<TaskResponseObject> getRunningTasks()
{
return getTasks("runningTasks");
}
public List<TaskResponseObject> getWaitingTasks()
{
return getTasks("waitingTasks");
}
public List<TaskResponseObject> getPendingTasks()
{
return getTasks("pendingTasks");
}
private List<TaskResponseObject> getTasks(String identifier)
{
try {
StatusResponseHolder response = makeRequest(
String.format("%s%s", getIndexerURL(), identifier)
);
LOG.info("Tasks %s response %s", identifier, response.getContent());
return jsonMapper.readValue(
response.getContent(), new TypeReference<List<TaskResponseObject>>()
{
}
);
}
catch (Exception e) {
throw Throwables.propagate(e);
}
}
public void waitUntilTaskCompletes(final String taskID)
{
RetryUtil.retryUntil(
new Callable<Boolean>()
{
@Override
public Boolean call() throws Exception
{
TaskStatus.Status status = getTaskStatus(taskID);
if (status == TaskStatus.Status.FAILED) {
throw new ISE("Indexer task FAILED");
}
return status == TaskStatus.Status.SUCCESS;
}
},
true,
60000,
10,
"Index Task to complete"
);
}
private StatusResponseHolder makeRequest(String url)
{
try {
StatusResponseHolder response = this.httpClient
.get(new URL(url))
.go(responseHandler)
.get();
if (!response.getStatus().equals(HttpResponseStatus.OK)) {
throw new ISE("Error while making request to indexer [%s %s]", response.getStatus(), response.getContent());
}
return response;
}
catch (Exception e) {
LOG.error(e, "Exception while sending request");
throw Throwables.propagate(e);
}
}
}

View File

@ -0,0 +1,98 @@
/*
* Druid - a distributed column store.
* Copyright (C) 2012, 2013 Metamarkets Group Inc.
*
* This program is free software; you can redistribute it and/or
* modify it under the terms of the GNU General Public License
* as published by the Free Software Foundation; either version 2
* of the License, or (at your option) any later version.
*
* This program is distributed in the hope that it will be useful,
* but WITHOUT ANY WARRANTY; without even the implied warranty of
* MERCHANTABILITY or FITNESS FOR A PARTICULAR PURPOSE. See the
* GNU General Public License for more details.
*
* You should have received a copy of the GNU General Public License
* along with this program; if not, write to the Free Software
* Foundation, Inc., 51 Franklin Street, Fifth Floor, Boston, MA 02110-1301, USA.
*/
package io.druid.testing.clients;
import com.fasterxml.jackson.core.type.TypeReference;
import com.fasterxml.jackson.databind.ObjectMapper;
import com.google.common.base.Charsets;
import com.google.common.base.Throwables;
import com.google.inject.Inject;
import com.metamx.common.ISE;
import com.metamx.http.client.HttpClient;
import com.metamx.http.client.response.StatusResponseHandler;
import com.metamx.http.client.response.StatusResponseHolder;
import io.druid.guice.annotations.Global;
import io.druid.query.Query;
import io.druid.testing.IntegrationTestingConfig;
import org.jboss.netty.handler.codec.http.HttpResponseStatus;
import java.net.URL;
import java.util.List;
import java.util.Map;
public class QueryResourceTestClient
{
private final ObjectMapper jsonMapper;
private final HttpClient httpClient;
private final String router;
private final StatusResponseHandler responseHandler;
@Inject
QueryResourceTestClient(
ObjectMapper jsonMapper,
@Global HttpClient httpClient,
IntegrationTestingConfig config
)
{
this.jsonMapper = jsonMapper;
this.httpClient = httpClient;
this.router = config.getRouterHost();
this.responseHandler = new StatusResponseHandler(Charsets.UTF_8);
}
private String getBrokerURL()
{
return String.format(
"http://%s/druid/v2/",
router
);
}
public List<Map<String, Object>> query(Query query)
{
try {
StatusResponseHolder response = httpClient.post(new URL(getBrokerURL()))
.setContent(
"application/json",
jsonMapper.writeValueAsBytes(query)
)
.go(responseHandler)
.get();
if (!response.getStatus().equals(HttpResponseStatus.OK)) {
throw new ISE(
"Error while querying[%s] status[%s] content[%s]",
getBrokerURL(),
response.getStatus(),
response.getContent()
);
}
return jsonMapper.readValue(
response.getContent(), new TypeReference<List<Map<String, Object>>>()
{
}
);
}
catch (Exception e) {
throw Throwables.propagate(e);
}
}
}

View File

@ -0,0 +1,68 @@
/*
* Druid - a distributed column store.
* Copyright (C) 2012, 2013 Metamarkets Group Inc.
*
* This program is free software; you can redistribute it and/or
* modify it under the terms of the GNU General Public License
* as published by the Free Software Foundation; either version 2
* of the License, or (at your option) any later version.
*
* This program is distributed in the hope that it will be useful,
* but WITHOUT ANY WARRANTY; without even the implied warranty of
* MERCHANTABILITY or FITNESS FOR A PARTICULAR PURPOSE. See the
* GNU General Public License for more details.
*
* You should have received a copy of the GNU General Public License
* along with this program; if not, write to the Free Software
* Foundation, Inc., 51 Franklin Street, Fifth Floor, Boston, MA 02110-1301, USA.
*/
package io.druid.testing.clients;
import com.fasterxml.jackson.annotation.JsonCreator;
import com.fasterxml.jackson.annotation.JsonProperty;
import io.druid.indexing.common.TaskStatus;
import org.joda.time.DateTime;
public class TaskResponseObject
{
private final String id;
private final DateTime createdTime;
private final DateTime queueInsertionTime;
private final TaskStatus status;
@JsonCreator
private TaskResponseObject(
@JsonProperty("id") String id,
@JsonProperty("createdTime") DateTime createdTime,
@JsonProperty("queueInsertionTime") DateTime queueInsertionTime,
@JsonProperty("status") TaskStatus status
)
{
this.id = id;
this.createdTime = createdTime;
this.queueInsertionTime = queueInsertionTime;
this.status = status;
}
public String getId()
{
return id;
}
public DateTime getCreatedTime()
{
return createdTime;
}
public DateTime getQueueInsertionTime()
{
return queueInsertionTime;
}
public TaskStatus getStatus()
{
return status;
}
}

View File

@ -0,0 +1,54 @@
/*
* Druid - a distributed column store.
* Copyright (C) 2012, 2013 Metamarkets Group Inc.
*
* This program is free software; you can redistribute it and/or
* modify it under the terms of the GNU General Public License
* as published by the Free Software Foundation; either version 2
* of the License, or (at your option) any later version.
*
* This program is distributed in the hope that it will be useful,
* but WITHOUT ANY WARRANTY; without even the implied warranty of
* MERCHANTABILITY or FITNESS FOR A PARTICULAR PURPOSE. See the
* GNU General Public License for more details.
*
* You should have received a copy of the GNU General Public License
* along with this program; if not, write to the Free Software
* Foundation, Inc., 51 Franklin Street, Fifth Floor, Boston, MA 02110-1301, USA.
*/
package io.druid.testing.guice;
import com.fasterxml.jackson.databind.ObjectMapper;
import com.google.common.base.Supplier;
import com.google.inject.Binder;
import com.google.inject.Module;
import com.google.inject.Provides;
import com.metamx.emitter.core.LoggingEmitter;
import com.metamx.emitter.core.LoggingEmitterConfig;
import com.metamx.emitter.service.ServiceEmitter;
import io.druid.guice.ConfigProvider;
import io.druid.guice.JsonConfigProvider;
import io.druid.guice.LazySingleton;
import io.druid.guice.ManageLifecycle;
import io.druid.testing.IntegrationTestingConfig;
import io.druid.testing.IntegrationTestingConfigProvider;
/**
*/
public class DruidTestModule implements Module
{
@Override
public void configure(Binder binder)
{
binder.bind(IntegrationTestingConfig.class).toProvider(IntegrationTestingConfigProvider.class).in(ManageLifecycle.class);
JsonConfigProvider.bind(binder, "druid.test.config", IntegrationTestingConfigProvider.class);
}
@Provides
@LazySingleton
public ServiceEmitter getServiceEmitter(Supplier<LoggingEmitterConfig> config, ObjectMapper jsonMapper)
{
return new ServiceEmitter("", "", new LoggingEmitter(config.get(), jsonMapper));
}
}

View File

@ -0,0 +1,63 @@
/*
* Druid - a distributed column store.
* Copyright (C) 2012, 2013 Metamarkets Group Inc.
*
* This program is free software; you can redistribute it and/or
* modify it under the terms of the GNU General Public License
* as published by the Free Software Foundation; either version 2
* of the License, or (at your option) any later version.
*
* This program is distributed in the hope that it will be useful,
* but WITHOUT ANY WARRANTY; without even the implied warranty of
* MERCHANTABILITY or FITNESS FOR A PARTICULAR PURPOSE. See the
* GNU General Public License for more details.
*
* You should have received a copy of the GNU General Public License
* along with this program; if not, write to the Free Software
* Foundation, Inc., 51 Franklin Street, Fifth Floor, Boston, MA 02110-1301, USA.
*/
package io.druid.testing.guice;
import com.google.common.collect.ImmutableList;
import com.google.inject.Injector;
import com.google.inject.Module;
import io.druid.guice.GuiceInjectors;
import io.druid.guice.IndexingServiceFirehoseModule;
import io.druid.initialization.Initialization;
import org.testng.IModuleFactory;
import org.testng.ITestContext;
import java.util.Collections;
import java.util.List;
public class DruidTestModuleFactory implements IModuleFactory
{
private static final Module module = new DruidTestModule();
private static final Injector injector = Initialization.makeInjectorWithModules(
GuiceInjectors.makeStartupInjector(),
getModules()
);
public static Injector getInjector()
{
return injector;
}
private static List<? extends Module> getModules()
{
return ImmutableList.of(
new DruidTestModule(),
new IndexingServiceFirehoseModule()
);
}
@Override
public Module createModule(ITestContext context, Class<?> testClass)
{
context.addGuiceModule(DruidTestModule.class, module);
context.addInjector(Collections.singletonList(module), injector);
return module;
}
}

View File

@ -0,0 +1,79 @@
/*
* Druid - a distributed column store.
* Copyright (C) 2012, 2013 Metamarkets Group Inc.
*
* This program is free software; you can redistribute it and/or
* modify it under the terms of the GNU General Public License
* as published by the Free Software Foundation; either version 2
* of the License, or (at your option) any later version.
*
* This program is distributed in the hope that it will be useful,
* but WITHOUT ANY WARRANTY; without even the implied warranty of
* MERCHANTABILITY or FITNESS FOR A PARTICULAR PURPOSE. See the
* GNU General Public License for more details.
*
* You should have received a copy of the GNU General Public License
* along with this program; if not, write to the Free Software
* Foundation, Inc., 51 Franklin Street, Fifth Floor, Boston, MA 02110-1301, USA.
*/
package io.druid.testing.utils;
import com.fasterxml.jackson.core.type.TypeReference;
import com.fasterxml.jackson.databind.ObjectMapper;
import com.google.inject.Inject;
import com.metamx.common.ISE;
import com.metamx.common.logger.Logger;
import io.druid.testing.clients.QueryResourceTestClient;
import java.util.List;
import java.util.Map;
public class FromFileTestQueryHelper
{
public static Logger LOG = new Logger(FromFileTestQueryHelper.class);
private final QueryResourceTestClient queryClient;
private final ObjectMapper jsonMapper;
@Inject
FromFileTestQueryHelper(ObjectMapper jsonMapper, QueryResourceTestClient queryClient)
{
this.jsonMapper = jsonMapper;
this.queryClient = queryClient;
}
public void testQueriesFromFile(String filePath, int timesToRun) throws Exception
{
LOG.info("Starting query tests for [%s]", filePath);
List<QueryWithResults> queries =
jsonMapper.readValue(
FromFileTestQueryHelper.class.getResourceAsStream(filePath),
new TypeReference<List<QueryWithResults>>()
{
}
);
for (int i = 0; i < timesToRun; i++) {
LOG.info("Starting Iteration " + i);
boolean failed = false;
for (QueryWithResults queryWithResult : queries) {
LOG.info("Running Query " + queryWithResult.getQuery().getType());
List<Map<String, Object>> result = queryClient.query(queryWithResult.getQuery());
if (!QueryResultVerifier.compareResults(result, queryWithResult.getExpectedResults())) {
LOG.error(
"Failed while executing %s actualResults : %s",
queryWithResult,
jsonMapper.writeValueAsString(result)
);
failed = true;
} else {
LOG.info("Results Verified for Query " + queryWithResult.getQuery().getType());
}
}
if (failed) {
throw new ISE("one or more twitter queries failed");
}
}
}
}

View File

@ -0,0 +1,49 @@
/*
* Druid - a distributed column store.
* Copyright (C) 2012, 2013 Metamarkets Group Inc.
*
* This program is free software; you can redistribute it and/or
* modify it under the terms of the GNU General Public License
* as published by the Free Software Foundation; either version 2
* of the License, or (at your option) any later version.
*
* This program is distributed in the hope that it will be useful,
* but WITHOUT ANY WARRANTY; without even the implied warranty of
* MERCHANTABILITY or FITNESS FOR A PARTICULAR PURPOSE. See the
* GNU General Public License for more details.
*
* You should have received a copy of the GNU General Public License
* along with this program; if not, write to the Free Software
* Foundation, Inc., 51 Franklin Street, Fifth Floor, Boston, MA 02110-1301, USA.
*/
package io.druid.testing.utils;
import java.util.Iterator;
import java.util.Map;
public class QueryResultVerifier
{
public static boolean compareResults(
Iterable<Map<String, Object>> actual,
Iterable<Map<String, Object>> expected
)
{
Iterator<Map<String, Object>> actualIter = actual.iterator();
Iterator<Map<String, Object>> expectedIter = expected.iterator();
while (actualIter.hasNext() && expectedIter.hasNext()) {
Map<String, Object> actualRes = actualIter.next();
Map<String, Object> expRes = expectedIter.next();
if (!actualRes.equals(expRes)) {
return false;
}
}
if (actualIter.hasNext() || expectedIter.hasNext()) {
return false;
}
return true;
}
}

View File

@ -0,0 +1,64 @@
/*
* Druid - a distributed column store.
* Copyright (C) 2012, 2013 Metamarkets Group Inc.
*
* This program is free software; you can redistribute it and/or
* modify it under the terms of the GNU General Public License
* as published by the Free Software Foundation; either version 2
* of the License, or (at your option) any later version.
*
* This program is distributed in the hope that it will be useful,
* but WITHOUT ANY WARRANTY; without even the implied warranty of
* MERCHANTABILITY or FITNESS FOR A PARTICULAR PURPOSE. See the
* GNU General Public License for more details.
*
* You should have received a copy of the GNU General Public License
* along with this program; if not, write to the Free Software
* Foundation, Inc., 51 Franklin Street, Fifth Floor, Boston, MA 02110-1301, USA.
*/
package io.druid.testing.utils;
import com.fasterxml.jackson.annotation.JsonCreator;
import com.fasterxml.jackson.annotation.JsonProperty;
import io.druid.query.Query;
import java.util.List;
import java.util.Map;
public class QueryWithResults
{
private final Query query;
private final List<Map<String, Object>> expectedResults;
@JsonCreator
public QueryWithResults(
@JsonProperty("query") Query query,
@JsonProperty("expectedResults") List<Map<String, Object>> expectedResults
)
{
this.query = query;
this.expectedResults = expectedResults;
}
@JsonProperty
public Query getQuery()
{
return query;
}
@JsonProperty
public List<Map<String, Object>> getExpectedResults()
{
return expectedResults;
}
@Override
public String toString()
{
return "QueryWithResults{" +
"query=" + query +
", expectedResults=" + expectedResults +
'}';
}
}

View File

@ -0,0 +1,76 @@
/*
* Druid - a distributed column store.
* Copyright (C) 2012, 2013 Metamarkets Group Inc.
*
* This program is free software; you can redistribute it and/or
* modify it under the terms of the GNU General Public License
* as published by the Free Software Foundation; either version 2
* of the License, or (at your option) any later version.
*
* This program is distributed in the hope that it will be useful,
* but WITHOUT ANY WARRANTY; without even the implied warranty of
* MERCHANTABILITY or FITNESS FOR A PARTICULAR PURPOSE. See the
* GNU General Public License for more details.
*
* You should have received a copy of the GNU General Public License
* along with this program; if not, write to the Free Software
* Foundation, Inc., 51 Franklin Street, Fifth Floor, Boston, MA 02110-1301, USA.
*/
package io.druid.testing.utils;
import com.google.common.base.Throwables;
import com.metamx.common.ISE;
import com.metamx.common.logger.Logger;
import java.util.concurrent.Callable;
import java.util.concurrent.TimeUnit;
public class RetryUtil
{
private static final Logger LOG = new Logger(RetryUtil.class);
public static int DEFAULT_RETRY_COUNT = 10;
public static long DEFAULT_RETRY_SLEEP = TimeUnit.SECONDS.toMillis(30);
public static void retryUntilTrue(Callable<Boolean> callable, String task)
{
retryUntil(callable, true, DEFAULT_RETRY_SLEEP, DEFAULT_RETRY_COUNT, task);
}
public static void retryUntilFalse(Callable<Boolean> callable, String task)
{
retryUntil(callable, false, DEFAULT_RETRY_SLEEP, DEFAULT_RETRY_COUNT, task);
}
public static void retryUntil(
Callable<Boolean> callable,
boolean expectedValue,
long delayInMillis,
int retryCount,
String taskMessage
)
{
try {
int currentTry = 0;
while (callable.call() != expectedValue) {
if (currentTry > retryCount) {
throw new ISE("Max number of retries[%d] exceeded for Task[%s]. Failing.", retryCount, taskMessage);
}
LOG.info(
"Attempt[%d]: Task %s still not complete. Next retry in %d ms",
currentTry, taskMessage, delayInMillis
);
Thread.sleep(delayInMillis);
currentTry++;
}
}
catch (Exception e) {
throw Throwables.propagate(e);
}
}
}

View File

@ -0,0 +1,64 @@
/*
* Druid - a distributed column store.
* Copyright (C) 2012, 2013 Metamarkets Group Inc.
*
* This program is free software; you can redistribute it and/or
* modify it under the terms of the GNU General Public License
* as published by the Free Software Foundation; either version 2
* of the License, or (at your option) any later version.
*
* This program is distributed in the hope that it will be useful,
* but WITHOUT ANY WARRANTY; without even the implied warranty of
* MERCHANTABILITY or FITNESS FOR A PARTICULAR PURPOSE. See the
* GNU General Public License for more details.
*
* You should have received a copy of the GNU General Public License
* along with this program; if not, write to the Free Software
* Foundation, Inc., 51 Franklin Street, Fifth Floor, Boston, MA 02110-1301, USA.
*/
package io.druid.testing.utils;
import com.metamx.common.logger.Logger;
import io.druid.client.selector.Server;
import io.druid.curator.discovery.ServerDiscoverySelector;
import java.util.concurrent.Callable;
public class ServerDiscoveryUtil
{
private static final Logger LOG = new Logger(ServerDiscoveryUtil.class);
public static boolean isInstanceReady(ServerDiscoverySelector serviceProvider)
{
try {
Server instance = serviceProvider.pick();
if (instance == null) {
LOG.warn("Unable to find a host");
return false;
}
}
catch (Exception e) {
LOG.error(e, "Caught exception waiting for host");
return false;
}
return true;
}
public static void waitUntilInstanceReady(final ServerDiscoverySelector serviceProvider, String instanceType)
{
RetryUtil.retryUntilTrue(
new Callable<Boolean>()
{
@Override
public Boolean call() throws Exception
{
return isInstanceReady(serviceProvider);
}
},
String.format("Instance %s to get ready", instanceType)
);
}
}

View File

@ -0,0 +1,150 @@
/*
* Druid - a distributed column store.
* Copyright (C) 2012, 2013 Metamarkets Group Inc.
*
* This program is free software; you can redistribute it and/or
* modify it under the terms of the GNU General Public License
* as published by the Free Software Foundation; either version 2
* of the License, or (at your option) any later version.
*
* This program is distributed in the hope that it will be useful,
* but WITHOUT ANY WARRANTY; without even the implied warranty of
* MERCHANTABILITY or FITNESS FOR A PARTICULAR PURPOSE. See the
* GNU General Public License for more details.
*
* You should have received a copy of the GNU General Public License
* along with this program; if not, write to the Free Software
* Foundation, Inc., 51 Franklin Street, Fifth Floor, Boston, MA 02110-1301, USA.
*/
package org.testng;
import com.google.api.client.repackaged.com.google.common.base.Throwables;
import com.google.api.client.util.Charsets;
import com.google.inject.Injector;
import com.google.inject.Key;
import com.metamx.common.lifecycle.Lifecycle;
import com.metamx.common.logger.Logger;
import com.metamx.http.client.HttpClient;
import com.metamx.http.client.response.StatusResponseHandler;
import com.metamx.http.client.response.StatusResponseHolder;
import io.druid.guice.annotations.Global;
import io.druid.testing.IntegrationTestingConfig;
import io.druid.testing.guice.DruidTestModuleFactory;
import io.druid.testing.utils.RetryUtil;
import org.jboss.netty.handler.codec.http.HttpResponseStatus;
import org.testng.internal.IConfiguration;
import org.testng.internal.annotations.IAnnotationFinder;
import org.testng.xml.XmlTest;
import java.net.URL;
import java.util.List;
import java.util.concurrent.Callable;
public class DruidTestRunnerFactory implements ITestRunnerFactory
{
private static final Logger LOG = new Logger(DruidTestRunnerFactory.class);
@Override
public TestRunner newTestRunner(
ISuite suite, XmlTest test, List<IInvokedMethodListener> listeners
)
{
IConfiguration configuration = TestNG.getDefault().getConfiguration();
String outputDirectory = suite.getOutputDirectory();
IAnnotationFinder annotationFinder = configuration.getAnnotationFinder();
Boolean skipFailedInvocationCounts = suite.getXmlSuite().skipFailedInvocationCounts();
return new DruidTestRunner(
configuration,
suite,
test,
outputDirectory,
annotationFinder,
skipFailedInvocationCounts,
listeners
);
}
private static class DruidTestRunner extends TestRunner
{
protected DruidTestRunner(
IConfiguration configuration,
ISuite suite,
XmlTest test,
String outputDirectory,
IAnnotationFinder finder,
boolean skipFailedInvocationCounts,
List<IInvokedMethodListener> invokedMethodListeners
)
{
super(configuration, suite, test, outputDirectory, finder, skipFailedInvocationCounts, invokedMethodListeners);
}
@Override
public void run()
{
Injector injector = DruidTestModuleFactory.getInjector();
IntegrationTestingConfig config = injector.getInstance(IntegrationTestingConfig.class);
HttpClient client = injector.getInstance(Key.get(HttpClient.class, Global.class));
;
waitUntilInstanceReady(client, config.getCoordinatorHost());
waitUntilInstanceReady(client, config.getIndexerHost());
waitUntilInstanceReady(client, config.getRouterHost());
Lifecycle lifecycle = injector.getInstance(Lifecycle.class);
try {
lifecycle.start();
runTests();
}
catch (Exception e) {
e.printStackTrace();
throw Throwables.propagate(e);
}
finally {
lifecycle.stop();
}
}
private void runTests()
{
super.run();
}
public void waitUntilInstanceReady(final HttpClient client, final String host)
{
final StatusResponseHandler handler = new StatusResponseHandler(Charsets.UTF_8);
RetryUtil.retryUntilTrue(
new Callable<Boolean>()
{
@Override
public Boolean call() throws Exception
{
try {
StatusResponseHolder response = client.get(
new URL(
String.format(
"http://%s/status",
host
)
)
)
.go(handler)
.get();
System.out.println(response.getStatus() + response.getContent());
if (response.getStatus().equals(HttpResponseStatus.OK)) {
return true;
} else {
return false;
}
}
catch (Throwable e) {
e.printStackTrace();
return false;
}
}
}, "Waiting for instance to be ready: [" + host + "]"
);
}
}
}

File diff suppressed because it is too large Load Diff

View File

@ -0,0 +1,336 @@
/*
* Druid - a distributed column store.
* Copyright (C) 2012, 2013 Metamarkets Group Inc.
*
* This program is free software; you can redistribute it and/or
* modify it under the terms of the GNU General Public License
* as published by the Free Software Foundation; either version 2
* of the License, or (at your option) any later version.
*
* This program is distributed in the hope that it will be useful,
* but WITHOUT ANY WARRANTY; without even the implied warranty of
* MERCHANTABILITY or FITNESS FOR A PARTICULAR PURPOSE. See the
* GNU General Public License for more details.
*
* You should have received a copy of the GNU General Public License
* along with this program; if not, write to the Free Software
* Foundation, Inc., 51 Franklin Street, Fifth Floor, Boston, MA 02110-1301, USA.
*/
package org.testng.remote;
import com.beust.jcommander.JCommander;
import com.beust.jcommander.ParameterException;
import org.testng.CommandLineArgs;
import org.testng.IInvokedMethodListener;
import org.testng.ISuite;
import org.testng.ISuiteListener;
import org.testng.ITestRunnerFactory;
import org.testng.TestNG;
import org.testng.TestNGException;
import org.testng.TestRunner;
import org.testng.collections.Lists;
import org.testng.internal.ClassHelper;
import org.testng.remote.strprotocol.GenericMessage;
import org.testng.remote.strprotocol.IMessageSender;
import org.testng.remote.strprotocol.MessageHelper;
import org.testng.remote.strprotocol.MessageHub;
import org.testng.remote.strprotocol.RemoteTestListener;
import org.testng.remote.strprotocol.SerializedMessageSender;
import org.testng.remote.strprotocol.StringMessageSender;
import org.testng.remote.strprotocol.SuiteMessage;
import org.testng.reporters.JUnitXMLReporter;
import org.testng.reporters.TestHTMLReporter;
import org.testng.xml.XmlSuite;
import org.testng.xml.XmlTest;
import java.util.Arrays;
import java.util.List;
import static org.testng.internal.Utils.defaultIfStringEmpty;
/**
* Class copied from TestNG library ver 6.8.7 to apply a workaround for http://jira.codehaus.org/browse/SUREFIRE-622
* To Locate the PATCHED AREA search for keyword "PATCH" in this class file
* <p/>
* Extension of TestNG registering a remote TestListener.
*
* @author Cedric Beust <cedric@beust.com>
*/
public class RemoteTestNG extends TestNG
{
// The following constants are referenced by the Eclipse plug-in, make sure you
// modify the plug-in as well if you change any of them.
public static final String DEBUG_PORT = "12345";
public static final String DEBUG_SUITE_FILE = "testng-customsuite.xml";
public static final String DEBUG_SUITE_DIRECTORY = System.getProperty("java.io.tmpdir");
public static final String PROPERTY_DEBUG = "testng.eclipse.debug";
public static final String PROPERTY_VERBOSE = "testng.eclipse.verbose";
private static final String LOCALHOST = "localhost";
// End of Eclipse constants.
/**
* Port used for the serialized protocol
*/
private static Integer m_serPort = null;
private static boolean m_debug;
private static boolean m_dontExit;
private static boolean m_ack;
private ITestRunnerFactory m_customTestRunnerFactory;
private String m_host;
/**
* Port used for the string protocol
*/
private Integer m_port = null;
public static void main(String[] args) throws ParameterException
{
CommandLineArgs cla = new CommandLineArgs();
RemoteArgs ra = new RemoteArgs();
new JCommander(Arrays.asList(cla, ra), args);
m_dontExit = ra.dontExit;
if (cla.port != null && ra.serPort != null) {
throw new TestNGException(
"Can only specify one of " + CommandLineArgs.PORT
+ " and " + RemoteArgs.PORT
);
}
m_debug = cla.debug;
m_ack = ra.ack;
if (m_debug) {
// while (true) {
initAndRun(args, cla, ra);
// }
} else {
initAndRun(args, cla, ra);
}
}
private static void initAndRun(String[] args, CommandLineArgs cla, RemoteArgs ra)
{
RemoteTestNG remoteTestNg = new RemoteTestNG();
if (m_debug) {
// In debug mode, override the port and the XML file to a fixed location
cla.port = Integer.parseInt(DEBUG_PORT);
ra.serPort = cla.port;
cla.suiteFiles = Arrays.asList(
new String[]{
DEBUG_SUITE_DIRECTORY + DEBUG_SUITE_FILE
}
);
}
remoteTestNg.configure(cla);
remoteTestNg.setHost(cla.host);
m_serPort = ra.serPort;
remoteTestNg.m_port = cla.port;
if (isVerbose()) {
StringBuilder sb = new StringBuilder("Invoked with ");
for (String s : args) {
sb.append(s).append(" ");
}
p(sb.toString());
// remoteTestNg.setVerbose(1);
// } else {
// remoteTestNg.setVerbose(0);
}
validateCommandLineParameters(cla);
remoteTestNg.run();
// if (m_debug) {
// // Run in a loop if in debug mode so it is possible to run several launches
// // without having to relauch RemoteTestNG.
// while (true) {
// remoteTestNg.run();
// remoteTestNg.configure(cla);
// }
// } else {
// remoteTestNg.run();
// }
}
private static void p(String s)
{
if (isVerbose()) {
System.out.println("[RemoteTestNG] " + s);
}
}
public static boolean isVerbose()
{
boolean result = System.getProperty(PROPERTY_VERBOSE) != null || isDebug();
return result;
}
public static boolean isDebug()
{
return m_debug || System.getProperty(PROPERTY_DEBUG) != null;
}
private void calculateAllSuites(List<XmlSuite> suites, List<XmlSuite> outSuites)
{
for (XmlSuite s : suites) {
outSuites.add(s);
// calculateAllSuites(s.getChildSuites(), outSuites);
}
}
@Override
public void run()
{
IMessageSender sender = m_serPort != null
? new SerializedMessageSender(m_host, m_serPort, m_ack)
: new StringMessageSender(m_host, m_port);
final MessageHub msh = new MessageHub(sender);
msh.setDebug(isDebug());
try {
msh.connect();
// We couldn't do this until now in debug mode since the .xml file didn't exist yet.
// Now that we have connected with the Eclipse client, we know that it created the .xml
// file so we can proceed with the initialization
initializeSuitesAndJarFile();
List<XmlSuite> suites = Lists.newArrayList();
calculateAllSuites(m_suites, suites);
// System.out.println("Suites: " + m_suites.get(0).getChildSuites().size()
// + " and:" + suites.get(0).getChildSuites().size());
if (suites.size() > 0) {
int testCount = 0;
for (int i = 0; i < suites.size(); i++) {
testCount += (suites.get(i)).getTests().size();
}
GenericMessage gm = new GenericMessage(MessageHelper.GENERIC_SUITE_COUNT);
gm.setSuiteCount(suites.size());
gm.setTestCount(testCount);
msh.sendMessage(gm);
addListener(new RemoteSuiteListener(msh));
setTestRunnerFactory(new DelegatingTestRunnerFactory(buildTestRunnerFactory(), msh));
// System.out.println("RemoteTestNG starting");
super.run();
} else {
System.err.println("No test suite found. Nothing to run");
}
}
catch (Throwable cause) {
cause.printStackTrace(System.err);
}
finally {
// System.out.println("RemoteTestNG finishing: " + (getEnd() - getStart()) + " ms");
msh.shutDown();
if (!m_debug && !m_dontExit) {
System.exit(0);
}
}
}
/**
* Override by the plugin if you need to configure differently the <code>TestRunner</code>
* (usually this is needed if different listeners/reporters are needed).
* <b>Note</b>: you don't need to worry about the wiring listener, because it is added
* automatically.
*/
protected ITestRunnerFactory buildTestRunnerFactory()
{
//################### PATCH STARTS
if (System.getProperty("testrunfactory") != null) {
m_customTestRunnerFactory = (ITestRunnerFactory) ClassHelper.newInstance(
ClassHelper.fileToClass(
System.getProperty(
"testrunfactory"
)
)
);
//################## PATCH ENDS
} else if (null == m_customTestRunnerFactory) {
m_customTestRunnerFactory = new ITestRunnerFactory()
{
@Override
public TestRunner newTestRunner(
ISuite suite, XmlTest xmlTest,
List<IInvokedMethodListener> listeners
)
{
TestRunner runner =
new TestRunner(
getConfiguration(), suite, xmlTest,
false /*skipFailedInvocationCounts */,
listeners
);
if (m_useDefaultListeners) {
runner.addListener(new TestHTMLReporter());
runner.addListener(new JUnitXMLReporter());
}
return runner;
}
};
}
return m_customTestRunnerFactory;
}
private String getHost()
{
return m_host;
}
public void setHost(String host)
{
m_host = defaultIfStringEmpty(host, LOCALHOST);
}
private int getPort()
{
return m_port;
}
/**
* A ISuiteListener wiring the results using the internal string-based protocol.
*/
private static class RemoteSuiteListener implements ISuiteListener
{
private final MessageHub m_messageSender;
RemoteSuiteListener(MessageHub smsh)
{
m_messageSender = smsh;
}
@Override
public void onFinish(ISuite suite)
{
m_messageSender.sendMessage(new SuiteMessage(suite, false /*start*/));
}
@Override
public void onStart(ISuite suite)
{
m_messageSender.sendMessage(new SuiteMessage(suite, true /*start*/));
}
}
private static class DelegatingTestRunnerFactory implements ITestRunnerFactory
{
private final ITestRunnerFactory m_delegateFactory;
private final MessageHub m_messageSender;
DelegatingTestRunnerFactory(ITestRunnerFactory trf, MessageHub smsh)
{
m_delegateFactory = trf;
m_messageSender = smsh;
}
@Override
public TestRunner newTestRunner(
ISuite suite, XmlTest test,
List<IInvokedMethodListener> listeners
)
{
TestRunner tr = m_delegateFactory.newTestRunner(suite, test, listeners);
tr.addListener(new RemoteTestListener(suite, test, m_messageSender));
return tr;
}
}
}

View File

@ -0,0 +1,85 @@
/*
* Druid - a distributed column store.
* Copyright (C) 2012, 2013 Metamarkets Group Inc.
*
* This program is free software; you can redistribute it and/or
* modify it under the terms of the GNU General Public License
* as published by the Free Software Foundation; either version 2
* of the License, or (at your option) any later version.
*
* This program is distributed in the hope that it will be useful,
* but WITHOUT ANY WARRANTY; without even the implied warranty of
* MERCHANTABILITY or FITNESS FOR A PARTICULAR PURPOSE. See the
* GNU General Public License for more details.
*
* You should have received a copy of the GNU General Public License
* along with this program; if not, write to the Free Software
* Foundation, Inc., 51 Franklin Street, Fifth Floor, Boston, MA 02110-1301, USA.
*/
package io.druid.tests.indexer;
import com.fasterxml.jackson.databind.ObjectMapper;
import com.google.inject.Inject;
import io.druid.testing.clients.CoordinatorResourceTestClient;
import io.druid.testing.clients.OverlordResourceTestClient;
import io.druid.testing.utils.FromFileTestQueryHelper;
import io.druid.testing.utils.RetryUtil;
import org.apache.commons.io.IOUtils;
import org.joda.time.Interval;
import java.io.IOException;
import java.io.InputStream;
import java.io.StringWriter;
import java.util.concurrent.Callable;
public abstract class AbstractIndexerTest
{
@Inject
protected CoordinatorResourceTestClient coordinator;
@Inject
protected OverlordResourceTestClient indexer;
@Inject
protected ObjectMapper jsonMapper;
@Inject
protected FromFileTestQueryHelper queryHelper;
protected void unloadAndKillData(final String dataSource) throws Exception
{
Interval interval = new Interval("2013-01-01T00:00:00.000Z/2013-12-01T00:00:00.000Z");
coordinator.unloadSegmentsForDataSource(dataSource, interval);
RetryUtil.retryUntilFalse(
new Callable<Boolean>()
{
@Override
public Boolean call() throws Exception
{
return coordinator.areSegmentsLoaded(dataSource);
}
}, "Segment Unloading"
);
coordinator.deleteSegmentsDataSource(dataSource, interval);
RetryUtil.retryUntilTrue(
new Callable<Boolean>()
{
@Override
public Boolean call() throws Exception
{
return (indexer.getPendingTasks().size() + indexer.getRunningTasks().size() + indexer.getWaitingTasks()
.size()) == 0;
}
}, "Waiting for Tasks Completion"
);
}
protected String getTaskAsString(String file) throws IOException
{
InputStream inputStream = ITRealtimeIndexTaskTest.class.getResourceAsStream(file);
StringWriter writer = new StringWriter();
IOUtils.copy(inputStream, writer, "UTF-8");
return writer.toString();
}
}

View File

@ -0,0 +1,81 @@
/*
* Druid - a distributed column store.
* Copyright (C) 2012, 2013 Metamarkets Group Inc.
*
* This program is free software; you can redistribute it and/or
* modify it under the terms of the GNU General Public License
* as published by the Free Software Foundation; either version 2
* of the License, or (at your option) any later version.
*
* This program is distributed in the hope that it will be useful,
* but WITHOUT ANY WARRANTY; without even the implied warranty of
* MERCHANTABILITY or FITNESS FOR A PARTICULAR PURPOSE. See the
* GNU General Public License for more details.
*
* You should have received a copy of the GNU General Public License
* along with this program; if not, write to the Free Software
* Foundation, Inc., 51 Franklin Street, Fifth Floor, Boston, MA 02110-1301, USA.
*/
package io.druid.tests.indexer;
import com.google.api.client.repackaged.com.google.common.base.Throwables;
import com.google.inject.Inject;
import com.metamx.common.logger.Logger;
import io.druid.testing.IntegrationTestingConfig;
import io.druid.testing.guice.DruidTestModuleFactory;
import io.druid.testing.utils.RetryUtil;
import org.testng.annotations.Guice;
import org.testng.annotations.Test;
import java.util.concurrent.Callable;
@Guice(moduleFactory = DruidTestModuleFactory.class)
public class ITIndexerTest extends AbstractIndexerTest
{
private static final Logger LOG = new Logger(ITIndexerTest.class);
private static String INDEX_TASK = "/indexer/wikipedia_index_task.json";
private static String INDEX_QUERIES_RESOURCE = "/indexer/wikipedia_index_queries.json";
private static String INDEX_DATASOURCE = "wikipedia_index_test";
@Inject
private IntegrationTestingConfig config;
@Test
public void testIndexData() throws Exception
{
loadData();
try {
queryHelper.testQueriesFromFile(INDEX_QUERIES_RESOURCE, 2);
}
catch (Exception e) {
e.printStackTrace();
Throwables.propagate(e);
}
finally {
unloadAndKillData(INDEX_DATASOURCE);
}
}
private void loadData() throws Exception
{
final String taskID = indexer.submitTask(getTaskAsString(INDEX_TASK));
LOG.info("TaskID for loading index task %s", taskID);
indexer.waitUntilTaskCompletes(taskID);
RetryUtil.retryUntilTrue(
new Callable<Boolean>()
{
@Override
public Boolean call() throws Exception
{
return coordinator.areSegmentsLoaded(INDEX_DATASOURCE);
}
}, "Segment Load"
);
}
}

View File

@ -0,0 +1,142 @@
/*
* Druid - a distributed column store.
* Copyright (C) 2012, 2013 Metamarkets Group Inc.
*
* This program is free software; you can redistribute it and/or
* modify it under the terms of the GNU General Public License
* as published by the Free Software Foundation; either version 2
* of the License, or (at your option) any later version.
*
* This program is distributed in the hope that it will be useful,
* but WITHOUT ANY WARRANTY; without even the implied warranty of
* MERCHANTABILITY or FITNESS FOR A PARTICULAR PURPOSE. See the
* GNU General Public License for more details.
*
* You should have received a copy of the GNU General Public License
* along with this program; if not, write to the Free Software
* Foundation, Inc., 51 Franklin Street, Fifth Floor, Boston, MA 02110-1301, USA.
*/
package io.druid.tests.indexer;
import com.google.api.client.repackaged.com.google.common.base.Throwables;
import com.google.inject.Inject;
import com.metamx.common.logger.Logger;
import com.metamx.http.client.HttpClient;
import io.druid.curator.discovery.ServerDiscoveryFactory;
import io.druid.curator.discovery.ServerDiscoverySelector;
import io.druid.guice.annotations.Global;
import io.druid.testing.IntegrationTestingConfig;
import io.druid.testing.clients.EventReceiverFirehoseTestClient;
import io.druid.testing.guice.DruidTestModuleFactory;
import io.druid.testing.utils.RetryUtil;
import io.druid.testing.utils.ServerDiscoveryUtil;
import org.joda.time.DateTime;
import org.testng.annotations.Guice;
import org.testng.annotations.Test;
import java.util.concurrent.Callable;
import java.util.concurrent.TimeUnit;
/**
* Steps
* 1) Submit a RealtimeIndexTask
* 2) Load Data using EventReceiverFirehose
* 3) Runs queries and verifies that the ingested data is available for queries
* 4) Waits for handover of the segment to historical node
* 5) Queries data from historical node and verifies handover
* 6) Removes and Delete the created Data Segment
*/
@Guice(moduleFactory = DruidTestModuleFactory.class)
public class ITRealtimeIndexTaskTest extends AbstractIndexerTest
{
private static final Logger LOG = new Logger(ITRealtimeIndexTaskTest.class);
private static final String REALTIME_TASK_RESOURCE = "/indexer/wikipedia_realtime_index_task.json";
private static final String EVENT_RECEIVER_SERVICE_NAME = "eventReceiverServiceName";
private static final String EVENT_DATA_FILE = "/indexer/wikipedia_index_data.json";
private static final String INDEX_QUERIES_RESOURCE = "/indexer/wikipedia_index_queries.json";
private static final String INDEX_DATASOURCE = "wikipedia_index_test";
@Inject
ServerDiscoveryFactory factory;
@Inject
@Global
HttpClient httpClient;
@Inject
IntegrationTestingConfig config;
@Test
public void testRealtimeIndexTask() throws Exception
{
try {
// the task will run for 3 minutes and then shutdown itself
String task = setShutOffTime(
getTaskAsString(REALTIME_TASK_RESOURCE),
new DateTime(System.currentTimeMillis() + TimeUnit.MINUTES.toMillis(2))
);
String taskID = indexer.submitTask(task);
postEvents();
// sleep for a while to let the events ingested
TimeUnit.SECONDS.sleep(5);
// should hit the queries on realtime task
this.queryHelper.testQueriesFromFile(INDEX_QUERIES_RESOURCE, 2);
// wait for the task to complete
indexer.waitUntilTaskCompletes(taskID);
// task should complete only after the segments are loaded by historical node
RetryUtil.retryUntil(
new Callable<Boolean>()
{
@Override
public Boolean call() throws Exception
{
return coordinator.areSegmentsLoaded(INDEX_DATASOURCE);
}
},
true,
60000,
10,
"Real-time generated segments loaded"
);
// run queries on historical nodes
this.queryHelper.testQueriesFromFile(INDEX_QUERIES_RESOURCE, 2);
}
catch (Exception e) {
e.printStackTrace();
Throwables.propagate(e);
}
finally {
unloadAndKillData(INDEX_DATASOURCE);
}
}
private String setShutOffTime(String taskAsString, DateTime time)
{
return taskAsString.replace("#SHUTOFFTIME", time.toString());
}
public void postEvents() throws Exception
{
final ServerDiscoverySelector eventReceiverSelector = factory.createSelector(EVENT_RECEIVER_SERVICE_NAME);
eventReceiverSelector.start();
try {
ServerDiscoveryUtil.waitUntilInstanceReady(eventReceiverSelector, "Event Receiver");
// Access the docker VM mapped host and port instead of service announced in zookeeper
String host = config.getMiddleManagerHost() + ":" + eventReceiverSelector.pick().getPort();
LOG.info("Event Receiver Found at host %s", host);
EventReceiverFirehoseTestClient client = new EventReceiverFirehoseTestClient(
host,
EVENT_RECEIVER_SERVICE_NAME,
jsonMapper,
httpClient
);
client.postEventsFromFile(EVENT_DATA_FILE);
}
finally {
eventReceiverSelector.stop();
}
}
}

View File

@ -0,0 +1,172 @@
/*
* Druid - a distributed column store.
* Copyright (C) 2012, 2013 Metamarkets Group Inc.
*
* This program is free software; you can redistribute it and/or
* modify it under the terms of the GNU General Public License
* as published by the Free Software Foundation; either version 2
* of the License, or (at your option) any later version.
*
* This program is distributed in the hope that it will be useful,
* but WITHOUT ANY WARRANTY; without even the implied warranty of
* MERCHANTABILITY or FITNESS FOR A PARTICULAR PURPOSE. See the
* GNU General Public License for more details.
*
* You should have received a copy of the GNU General Public License
* along with this program; if not, write to the Free Software
* Foundation, Inc., 51 Franklin Street, Fifth Floor, Boston, MA 02110-1301, USA.
*/
package io.druid.tests.indexer;
import com.beust.jcommander.internal.Lists;
import com.google.api.client.repackaged.com.google.common.base.Throwables;
import com.google.inject.Inject;
import com.metamx.common.logger.Logger;
import com.metamx.http.client.HttpClient;
import io.druid.curator.discovery.ServerDiscoveryFactory;
import io.druid.curator.discovery.ServerDiscoverySelector;
import io.druid.guice.annotations.Global;
import io.druid.testing.IntegrationTestingConfig;
import io.druid.testing.clients.EventReceiverFirehoseTestClient;
import io.druid.testing.guice.DruidTestModuleFactory;
import io.druid.testing.utils.RetryUtil;
import io.druid.testing.utils.ServerDiscoveryUtil;
import org.joda.time.DateTime;
import org.testng.annotations.Guice;
import org.testng.annotations.Test;
import java.util.List;
import java.util.concurrent.Callable;
import java.util.concurrent.TimeUnit;
@Guice(moduleFactory = DruidTestModuleFactory.class)
public class ITUnionQueryTest extends AbstractIndexerTest
{
private static final Logger LOG = new Logger(ITUnionQueryTest.class);
private static final String REALTIME_TASK_RESOURCE = "/indexer/wikipedia_realtime_index_task.json";
private static final String EVENT_RECEIVER_SERVICE_PREFIX = "eventReceiverServiceName";
private static final String UNION_DATA_FILE = "/indexer/wikipedia_index_data.json";
private static final String UNION_QUERIES_RESOURCE = "/indexer/union_queries.json";
private static final String UNION_DATASOURCE = "wikipedia_index_test";
@Inject
ServerDiscoveryFactory factory;
@Inject
@Global
HttpClient httpClient;
@Inject
IntegrationTestingConfig config;
@Test
public void testRealtimeIndexTask() throws Exception
{
final int numTasks = 4;
try {
// Load 4 datasources with same dimensions
String task = setShutOffTime(
getTaskAsString(REALTIME_TASK_RESOURCE),
new DateTime(System.currentTimeMillis() + TimeUnit.MINUTES.toMillis(3))
);
List<String> taskIDs = Lists.newArrayList();
for (int i = 0; i < numTasks; i++) {
taskIDs.add(
indexer.submitTask(
withServiceName(
withDataSource(task, UNION_DATASOURCE + i),
EVENT_RECEIVER_SERVICE_PREFIX + i
)
)
);
}
for (int i = 0; i < numTasks; i++) {
postEvents(i);
}
// sleep for a while to let the events ingested
TimeUnit.SECONDS.sleep(5);
// should hit the queries on realtime task
LOG.info("Running Union Queries..");
this.queryHelper.testQueriesFromFile(UNION_QUERIES_RESOURCE, 2);
// wait for the task to complete
for (int i = 0; i < numTasks; i++) {
indexer.waitUntilTaskCompletes(taskIDs.get(i));
}
// task should complete only after the segments are loaded by historical node
for (int i = 0; i < numTasks; i++) {
final int taskNum = i;
RetryUtil.retryUntil(
new Callable<Boolean>()
{
@Override
public Boolean call() throws Exception
{
return coordinator.areSegmentsLoaded(UNION_DATASOURCE + taskNum);
}
},
true,
60000,
10,
"Real-time generated segments loaded"
);
}
// run queries on historical nodes
this.queryHelper.testQueriesFromFile(UNION_QUERIES_RESOURCE, 2);
}
catch (Exception e) {
e.printStackTrace();
throw Throwables.propagate(e);
}
finally {
for (int i = 0; i < numTasks; i++) {
unloadAndKillData(UNION_DATASOURCE + i);
}
}
}
private String setShutOffTime(String taskAsString, DateTime time)
{
return taskAsString.replace("#SHUTOFFTIME", time.toString());
}
private String withDataSource(String taskAsString, String dataSource)
{
return taskAsString.replace(UNION_DATASOURCE, dataSource);
}
private String withServiceName(String taskAsString, String serviceName)
{
return taskAsString.replace(EVENT_RECEIVER_SERVICE_PREFIX, serviceName);
}
public void postEvents(int id) throws Exception
{
final ServerDiscoverySelector eventReceiverSelector = factory.createSelector(EVENT_RECEIVER_SERVICE_PREFIX + id);
eventReceiverSelector.start();
try {
ServerDiscoveryUtil.waitUntilInstanceReady(eventReceiverSelector, "Event Receiver");
// Access the docker VM mapped host and port instead of service announced in zookeeper
String host = config.getMiddleManagerHost() + ":" + eventReceiverSelector.pick().getPort();
LOG.info("Event Receiver Found at host [%s]", host);
EventReceiverFirehoseTestClient client = new EventReceiverFirehoseTestClient(
host,
EVENT_RECEIVER_SERVICE_PREFIX + id,
jsonMapper,
httpClient
);
client.postEventsFromFile(UNION_DATA_FILE);
}
finally {
eventReceiverSelector.stop();
}
}
}

View File

@ -0,0 +1,65 @@
/*
* Druid - a distributed column store.
* Copyright (C) 2012, 2013 Metamarkets Group Inc.
*
* This program is free software; you can redistribute it and/or
* modify it under the terms of the GNU General Public License
* as published by the Free Software Foundation; either version 2
* of the License, or (at your option) any later version.
*
* This program is distributed in the hope that it will be useful,
* but WITHOUT ANY WARRANTY; without even the implied warranty of
* MERCHANTABILITY or FITNESS FOR A PARTICULAR PURPOSE. See the
* GNU General Public License for more details.
*
* You should have received a copy of the GNU General Public License
* along with this program; if not, write to the Free Software
* Foundation, Inc., 51 Franklin Street, Fifth Floor, Boston, MA 02110-1301, USA.
*/
package io.druid.tests.query;
import com.google.inject.Inject;
import io.druid.testing.clients.CoordinatorResourceTestClient;
import io.druid.testing.guice.DruidTestModuleFactory;
import io.druid.testing.utils.FromFileTestQueryHelper;
import io.druid.testing.utils.RetryUtil;
import org.testng.annotations.BeforeMethod;
import org.testng.annotations.Guice;
import org.testng.annotations.Test;
import java.util.concurrent.Callable;
@Guice(moduleFactory = DruidTestModuleFactory.class)
public class ITTwitterQueryTest
{
private static final String TWITTER_DATA_SOURCE = "twitterstream";
private static final String TWITTER_QUERIES_RESOURCE = "/queries/twitterstream_queries.json";
@Inject
CoordinatorResourceTestClient coordinatorClient;
@Inject
private FromFileTestQueryHelper queryHelper;
@BeforeMethod
public void before()
{
// ensure that the segments twitter segments are loaded completely
RetryUtil.retryUntilTrue(
new Callable<Boolean>()
{
@Override
public Boolean call() throws Exception
{
return coordinatorClient.areSegmentsLoaded(TWITTER_DATA_SOURCE);
}
}, "twitter segment load"
);
}
@Test
public void testQueriesFromFile() throws Exception
{
queryHelper.testQueriesFromFile(TWITTER_QUERIES_RESOURCE, 2);
}
}

View File

@ -0,0 +1,65 @@
/*
* Druid - a distributed column store.
* Copyright (C) 2012, 2013 Metamarkets Group Inc.
*
* This program is free software; you can redistribute it and/or
* modify it under the terms of the GNU General Public License
* as published by the Free Software Foundation; either version 2
* of the License, or (at your option) any later version.
*
* This program is distributed in the hope that it will be useful,
* but WITHOUT ANY WARRANTY; without even the implied warranty of
* MERCHANTABILITY or FITNESS FOR A PARTICULAR PURPOSE. See the
* GNU General Public License for more details.
*
* You should have received a copy of the GNU General Public License
* along with this program; if not, write to the Free Software
* Foundation, Inc., 51 Franklin Street, Fifth Floor, Boston, MA 02110-1301, USA.
*/
package io.druid.tests.query;
import com.google.inject.Inject;
import io.druid.testing.clients.CoordinatorResourceTestClient;
import io.druid.testing.guice.DruidTestModuleFactory;
import io.druid.testing.utils.FromFileTestQueryHelper;
import io.druid.testing.utils.RetryUtil;
import org.testng.annotations.BeforeMethod;
import org.testng.annotations.Guice;
import org.testng.annotations.Test;
import java.util.concurrent.Callable;
@Guice(moduleFactory = DruidTestModuleFactory.class)
public class ITWikipediaQueryTest
{
private static final String WIKIPEDIA_DATA_SOURCE = "wikipedia_editstream";
private static final String WIKIPEDIA_QUERIES_RESOURCE = "/queries/wikipedia_editstream_queries.json";
@Inject
private CoordinatorResourceTestClient coordinatorClient;
@Inject
private FromFileTestQueryHelper queryHelper;
@BeforeMethod
public void before()
{
// ensure that twitter segments are loaded completely
RetryUtil.retryUntilTrue(
new Callable<Boolean>()
{
@Override
public Boolean call() throws Exception
{
return coordinatorClient.areSegmentsLoaded(WIKIPEDIA_DATA_SOURCE);
}
}, "wikipedia segment load"
);
}
@Test
public void testQueriesFromFile() throws Exception
{
queryHelper.testQueriesFromFile(WIKIPEDIA_QUERIES_RESOURCE, 2);
}
}

View File

@ -0,0 +1,19 @@
{
"queryType": "select",
"intervals": ["2013-08-31/2013-09-01"],
"dataSource": "wikipedia_index_test",
"granularity": "all",
"filter": {
"type": "selector",
"dimension": "language",
"value": "en"
},
"pagingSpec": {
"threshold": 10
},
"context": {
"useCache": "false",
"populateCache": "false",
"timeout": 60000
}
}

View File

@ -0,0 +1,19 @@
{
"queryType": "select",
"intervals": ["2013-08-31/2013-09-01"],
"dataSource": "wikipedia_reindex_test",
"granularity": "all",
"filter": {
"type": "selector",
"dimension": "language",
"value": "en"
},
"pagingSpec": {
"threshold": 10
},
"context": {
"useCache": "false",
"populateCache": "false",
"timeout": 60000
}
}

View File

@ -0,0 +1,564 @@
[
{
"description": "timeseries, filtered, all aggs, all",
"query": {
"queryType": "timeseries",
"dataSource": {
"type": "union",
"dataSources": [
"wikipedia_index_test1", "wikipedia_index_test2", "wikipedia_index_test3",
"wikipedia_index_test0"
]
},
"intervals": ["2013-08-31/2013-09-01"],
"granularity": "all",
"filter": {
"type": "selector",
"dimension": "language",
"value": "en"
},
"aggregations": [
{
"type": "count",
"name": "rows"
},
{
"type": "longSum",
"fieldName": "count",
"name": "count"
},
{
"type": "doubleSum",
"fieldName": "added",
"name": "added"
},
{
"type": "doubleSum",
"fieldName": "deleted",
"name": "deleted"
},
{
"type": "doubleSum",
"fieldName": "delta",
"name": "delta"
}
],
"context": {
"useCache": "true",
"populateCache": "true",
"timeout": 60000
}
},
"expectedResults": [
{
"timestamp": "2013-08-31T01:02:33.000Z",
"result": {
"added": 2064.0,
"count": 8,
"delta": 748.0,
"deleted": 1316.0,
"rows": 8
}
}
]
},
{
"description": "topN, all aggs, page dim, uniques metric",
"query": {
"queryType": "topN",
"dataSource": {
"type": "union",
"dataSources": [
"wikipedia_index_test1", "wikipedia_index_test2", "wikipedia_index_test3",
"wikipedia_index_test0"
]
},
"intervals": ["2013-08-31/2013-09-01"],
"granularity": "all",
"aggregations": [
{
"type": "count",
"name": "rows"
},
{
"type": "longSum",
"fieldName": "count",
"name": "count"
},
{
"type": "doubleSum",
"fieldName": "added",
"name": "added"
},
{
"type": "doubleSum",
"fieldName": "deleted",
"name": "deleted"
},
{
"type": "doubleSum",
"fieldName": "delta",
"name": "delta"
}
],
"dimension": "page",
"metric": "added",
"threshold": 3,
"context": {
"useCache": "true",
"populateCache": "true",
"timeout": 60000
}
},
"expectedResults": [
{
"timestamp": "2013-08-31T01:02:33.000Z",
"result": [
{
"added": 3620.0,
"count": 4,
"page": "Crimson Typhoon",
"delta": 3600.0,
"deleted": 20.0,
"rows": 4
},
{
"added": 1836.0,
"count": 4,
"page": "Striker Eureka",
"delta": 1320.0,
"deleted": 516.0,
"rows": 4
},
{
"added": 492.0,
"count": 4,
"page": "Cherno Alpha",
"delta": 444.0,
"deleted": 48.0,
"rows": 4
}
]
}
]
},
{
"description": "topN, all aggs, page dim, count metric, postAggs",
"query": {
"queryType": "topN",
"dataSource": {
"type": "union",
"dataSources": [
"wikipedia_index_test1", "wikipedia_index_test2", "wikipedia_index_test3",
"wikipedia_index_test0"
]
},
"intervals": ["2013-08-31/2013-09-01"],
"granularity": "all",
"aggregations": [
{
"type": "count",
"name": "rows"
},
{
"type": "longSum",
"fieldName": "count",
"name": "count"
},
{
"type": "doubleSum",
"fieldName": "added",
"name": "added"
},
{
"type": "doubleSum",
"fieldName": "deleted",
"name": "deleted"
},
{
"type": "doubleSum",
"fieldName": "delta",
"name": "delta"
}
],
"postAggregations": [
{
"type": "arithmetic",
"name": "sumOfAddedDeletedConst",
"fn": "+",
"fields": [
{
"type": "fieldAccess",
"name": "added",
"fieldName": "added"
},
{
"type": "arithmetic",
"name": "",
"fn": "+",
"fields": [
{
"type": "fieldAccess",
"name": "deleted",
"fieldName": "deleted"
},
{
"type": "constant",
"name": "constant",
"value": 1000
}
]
}
]
}
],
"dimension": "page",
"metric": "added",
"threshold": 3,
"context": {
"useCache": "true",
"populateCache": "true",
"timeout": 60000
}
},
"expectedResults": [
{
"timestamp": "2013-08-31T01:02:33.000Z",
"result": [
{
"added": 3620.0,
"count": 4,
"page": "Crimson Typhoon",
"delta": 3600.0,
"deleted": 20.0,
"sumOfAddedDeletedConst": 4640.0,
"rows": 4
},
{
"added": 1836.0,
"count": 4,
"page": "Striker Eureka",
"delta": 1320.0,
"deleted": 516.0,
"sumOfAddedDeletedConst": 3352.0,
"rows": 4
},
{
"added": 492.0,
"count": 4,
"page": "Cherno Alpha",
"delta": 444.0,
"deleted": 48.0,
"sumOfAddedDeletedConst": 1540.0,
"rows": 4
}
]
}
]
},
{
"description": "topN, lexicographic, two aggs, language dim, postAggs",
"query": {
"queryType": "topN",
"dataSource": {
"type": "union",
"dataSources": [
"wikipedia_index_test1", "wikipedia_index_test2", "wikipedia_index_test3",
"wikipedia_index_test0"
]
},
"intervals": ["2013-08-31/2013-09-01"],
"granularity": "all",
"aggregations": [
{
"type": "count",
"name": "rows"
},
{
"type": "longSum",
"fieldName": "count",
"name": "count"
}
],
"postAggregations": [
{
"type": "arithmetic",
"name": "sumOfRowsAndCount",
"fn": "+",
"fields": [
{
"type": "fieldAccess",
"name": "rows",
"fieldName": "rows"
},
{
"type": "fieldAccess",
"name": "count",
"fieldName": "count"
}
]
}
],
"dimension": "language",
"metric": {
"type": "lexicographic",
"previousStop": "a"
},
"threshold": 3,
"context": {
"useCache": "true",
"populateCache": "true",
"timeout": 60000
}
},
"expectedResults": [
{
"timestamp": "2013-08-31T01:02:33.000Z",
"result": [
{
"sumOfRowsAndCount": 16.0,
"count": 8,
"language": "en",
"rows": 8
},
{
"sumOfRowsAndCount": 8.0,
"count": 4,
"language": "ja",
"rows": 4
},
{
"sumOfRowsAndCount": 8.0,
"count": 4,
"language": "ru",
"rows": 4
}
]
}
]
},
{
"description": "groupBy, two aggs, namespace dim, postAggs",
"query": {
"queryType": "groupBy",
"dataSource": {
"type": "union",
"dataSources": [
"wikipedia_index_test1", "wikipedia_index_test2", "wikipedia_index_test3",
"wikipedia_index_test0"
]
},
"intervals": ["2013-08-31/2013-09-01"],
"granularity": "all",
"aggregations": [
{
"type": "count",
"name": "rows"
},
{
"type": "longSum",
"fieldName": "count",
"name": "count"
}
],
"postAggregations": [
{
"type": "arithmetic",
"name": "sumOfRowsAndCount",
"fn": "+",
"fields": [
{
"type": "fieldAccess",
"name": "rows",
"fieldName": "rows"
},
{
"type": "fieldAccess",
"name": "count",
"fieldName": "count"
}
]
}
],
"dimensions": ["namespace"],
"context": {
"useCache": "true",
"populateCache": "true",
"timeout": 60000
}
},
"expectedResults": [
{
"version": "v1",
"timestamp": "2013-08-31T00:00:00.000Z",
"event": {
"sumOfRowsAndCount": 16.0,
"count": 8,
"rows": 8,
"namespace": "article"
}
},
{
"version": "v1",
"timestamp": "2013-08-31T00:00:00.000Z",
"event": {
"sumOfRowsAndCount": 24.0,
"count": 12,
"rows": 12,
"namespace": "wikipedia"
}
}
]
},
{
"description": "groupBy, two aggs, namespace + robot dim, postAggs",
"query": {
"queryType": "groupBy",
"dataSource": {
"type": "union",
"dataSources": [
"wikipedia_index_test1", "wikipedia_index_test2", "wikipedia_index_test3",
"wikipedia_index_test0"
]
},
"intervals": ["2013-08-31/2013-09-01"],
"granularity": "all",
"aggregations": [
{
"type": "count",
"name": "rows"
},
{
"type": "longSum",
"fieldName": "count",
"name": "count"
}
],
"postAggregations": [
{
"type": "arithmetic",
"name": "sumOfRowsAndCount",
"fn": "+",
"fields": [
{
"type": "fieldAccess",
"name": "rows",
"fieldName": "rows"
},
{
"type": "fieldAccess",
"name": "count",
"fieldName": "count"
}
]
}
],
"dimensions": ["namespace", "robot"],
"limitSpec": {
"type": "default",
"limit": 3,
"orderBy": ["robot", "namespace"]
},
"context": {
"useCache": "true",
"populateCache": "true",
"timeout": 60000
}
},
"expectedResults": [
{
"version": "v1",
"timestamp": "2013-08-31T00:00:00.000Z",
"event": {
"sumOfRowsAndCount": 8.0,
"count": 4,
"robot": "false",
"rows": 4,
"namespace": "article"
}
},
{
"version": "v1",
"timestamp": "2013-08-31T00:00:00.000Z",
"event": {
"sumOfRowsAndCount": 8.0,
"count": 4,
"robot": "true",
"rows": 4,
"namespace": "article"
}
},
{
"version": "v1",
"timestamp": "2013-08-31T00:00:00.000Z",
"event": {
"sumOfRowsAndCount": 24.0,
"count": 12,
"robot": "true",
"rows": 12,
"namespace": "wikipedia"
}
}
]
},
{
"query": {
"queryType": "search",
"intervals": ["2013-08-31/2013-09-01"],
"dataSource": {
"type": "union",
"dataSources": [
"wikipedia_index_test1", "wikipedia_index_test2", "wikipedia_index_test3",
"wikipedia_index_test0"
]
},
"granularity": "all",
"query": {
"type": "insensitive_contains",
"value": "ip"
},
"context": {
"useCache": "true",
"populateCache": "true",
"timeout": 60000
}
},
"expectedResults": [
{
"timestamp": "2013-08-31T00:00:00.000Z",
"result": [
{
"dimension": "user",
"value": "triplets"
},
{
"dimension": "namespace",
"value": "wikipedia"
}
]
}
]
},
{
"description": "timeboundary, 1 agg, union",
"query": {
"queryType": "timeBoundary",
"dataSource": {
"type": "union",
"dataSources": [
"wikipedia_index_test1", "wikipedia_index_test2", "wikipedia_index_test3",
"wikipedia_index_test0"
]
}
},
"expectedResults": [
{
"timestamp": "2013-08-31T01:02:33.000Z",
"result": {
"minTime": "2013-08-31T01:02:33.000Z",
"maxTime": "2013-08-31T12:41:27.000Z"
}
}
]
}
]

View File

@ -0,0 +1,121 @@
{
"query": {
"queryType": "select",
"intervals": ["2013-08-31/2013-09-01"],
"dataSource": {
"type": "union",
"dataSources": [
"wikipedia_index_test"
]
},
"granularity": "all",
"filter": {
"type": "selector",
"dimension": "language",
"value": "en"
},
"pagingSpec": {
"threshold": 10
},
"context": {
"useCache": "false",
"populateCache": "false",
"timeout": 60000
}
},
"expectedResults": [
{
"timestamp": "2013-08-31T01:02:33.000Z",
"result": {
"pagingIdentifiers": {
"wikipedia_index_test0_2013-08-31T00:00:00.000Z_2013-09-01T00:00:00.000Z_2014-05-01T15:27:43.993Z": 0,
"wikipedia_index_test1_2013-08-31T00:00:00.000Z_2013-09-01T00:00:00.000Z_2014-05-01T15:27:44.108Z": 0,
"wikipedia_index_test2_2013-08-31T00:00:00.000Z_2013-09-01T00:00:00.000Z_2014-05-01T15:27:44.236Z": 0,
"wikipedia_index_test3_2013-08-31T00:00:00.000Z_2013-09-01T00:00:00.000Z_2014-05-01T15:27:44.374Z": 0
},
"events": [
{
"segmentId": "wikipedia_index_test0_2013-08-31T00:00:00.000Z_2013-09-01T00:00:00.000Z_2014-05-01T15:27:43.993Z",
"offset": 0,
"event": {
"timestamp": "2013-08-31T01:02:33.000Z",
"page": "Gypsy Danger",
"added": 57.0,
"deleted": 200.0
}
},
{
"segmentId": "wikipedia_index_test1_2013-08-31T00:00:00.000Z_2013-09-01T00:00:00.000Z_2014-05-01T15:27:44.108Z",
"offset": 0,
"event": {
"timestamp": "2013-08-31T01:02:33.000Z",
"page": "Gypsy Danger",
"added": 57.0,
"deleted": 200.0
}
},
{
"segmentId": "wikipedia_index_test2_2013-08-31T00:00:00.000Z_2013-09-01T00:00:00.000Z_2014-05-01T15:27:44.236Z",
"offset": 0,
"event": {
"timestamp": "2013-08-31T01:02:33.000Z",
"page": "Gypsy Danger",
"added": 57.0,
"deleted": 200.0
}
},
{
"segmentId": "wikipedia_index_test3_2013-08-31T00:00:00.000Z_2013-09-01T00:00:00.000Z_2014-05-01T15:27:44.374Z",
"offset": 0,
"event": {
"timestamp": "2013-08-31T01:02:33.000Z",
"page": "Gypsy Danger",
"added": 57.0,
"deleted": 200.0
}
},
{
"segmentId": "wikipedia_index_test0_2013-08-31T0com.metamx.common.ISE: one or more twitter queries failed0:00:00.000Z_2013-09-01T00:00:00.000Z_2014-05-01T15:27:43.993Z",
"offset": 0,
"event": {
"timestamp": "2013-08-31T03:32:45.000Z",
"page": "Striker Eureka",
"added": 459.0,
"deleted": 129.0
}
},
{
"segmentId": "wikipedia_index_test1_2013-08-31T00:00:00.000Z_2013-09-01T00:00:00.000Z_2014-05-01T15:27:44.108Z",
"offset": 0,
"event": {
"timestamp": "2013-08-31T03:32:45.000Z",
"page": "Striker Eureka",
"added": 459.0,
"deleted": 129.0
}
},
{
"segmentId": "wikipedia_index_test2_2013-08-31T00:00:00.000Z_2013-09-01T00:00:00.000Z_2014-05-01T15:27:44.236Z",
"offset": 0,
"event": {
"timestamp": "2013-08-31T03:32:45.000Z",
"page": "Striker Eureka",
"added": 459.0,
"deleted": 129.0
}
},
{
"segmentId": "wikipedia_index_test3_2013-08-31T00:00:00.000Z_2013-09-01T00:00:00.000Z_2014-05-01T15:27:44.374Z",
"offset": 0,
"event": {
"timestamp": "2013-08-31T03:32:45.000Z",
"page": "Striker Eureka",
"added": 459.0,
"deleted": 129.0
}
}
]
}
}
]
}

View File

@ -0,0 +1,5 @@
{"timestamp": "2013-08-31T01:02:33Z", "page": "Gypsy Danger", "language" : "en", "user" : "nuclear", "unpatrolled" : "true", "newPage" : "true", "robot": "false", "anonymous": "false", "namespace":"article", "continent":"North America", "country":"United States", "region":"Bay Area", "city":"San Francisco", "added": 57, "deleted": 200, "delta": -143}
{"timestamp": "2013-08-31T03:32:45Z", "page": "Striker Eureka", "language" : "en", "user" : "speed", "unpatrolled" : "false", "newPage" : "true", "robot": "true", "anonymous": "false", "namespace":"wikipedia", "continent":"Australia", "country":"Australia", "region":"Cantebury", "city":"Syndey", "added": 459, "deleted": 129, "delta": 330}
{"timestamp": "2013-08-31T07:11:21Z", "page": "Cherno Alpha", "language" : "ru", "user" : "masterYi", "unpatrolled" : "false", "newPage" : "true", "robot": "true", "anonymous": "false", "namespace":"article", "continent":"Asia", "country":"Russia", "region":"Oblast", "city":"Moscow", "added": 123, "deleted": 12, "delta": 111}
{"timestamp": "2013-08-31T11:58:39Z", "page": "Crimson Typhoon", "language" : "zh", "user" : "triplets", "unpatrolled" : "true", "newPage" : "false", "robot": "true", "anonymous": "false", "namespace":"wikipedia", "continent":"Asia", "country":"China", "region":"Shanxi", "city":"Taiyuan", "added": 905, "deleted": 5, "delta": 900}
{"timestamp": "2013-08-31T12:41:27Z", "page": "Coyote Tango", "language" : "ja", "user" : "stringer", "unpatrolled" : "true", "newPage" : "false", "robot": "true", "anonymous": "false", "namespace":"wikipedia", "continent":"Asia", "country":"Japan", "region":"Kanto", "city":"Tokyo", "added": 1, "deleted": 10, "delta": -9}

View File

@ -0,0 +1,16 @@
[
{
"description": "timeseries, 1 agg, all",
"query":{
"queryType" : "timeBoundary",
"dataSource": "wikipedia_index_test"
},
"expectedResults":[ {
"timestamp" : "2013-08-31T01:02:33.000Z",
"result" : {
"minTime" : "2013-08-31T01:02:33.000Z",
"maxTime" : "2013-08-31T12:41:27.000Z"
}
} ]
}
]

View File

@ -0,0 +1,59 @@
{
"type": "index",
"spec": {
"dataSchema": {
"dataSource": "wikipedia_index_test",
"metricsSpec": [
{
"type": "count",
"name": "count"
},
{
"type": "doubleSum",
"name": "added",
"fieldName": "added"
},
{
"type": "doubleSum",
"name": "deleted",
"fieldName": "deleted"
},
{
"type": "doubleSum",
"name": "delta",
"fieldName": "delta"
}
],
"granularitySpec": {
"segmentGranularity": "DAY",
"queryGranularity": "second",
"intervals" : [ "2013-08-31/2013-09-01" ]
},
"parser": {
"parseSpec": {
"format" : "json",
"timestampSpec": {
"column": "timestamp"
},
"dimensionsSpec": {
"dimensions": [
"page", "language", "user", "unpatrolled", "newPage", "robot", "anonymous",
"namespace", "continent", "country", "region", "city"
]
}
}
}
},
"ioConfig": {
"type": "index",
"firehose": {
"type": "local",
"baseDir": "/resources/indexer",
"filter": "wikipedia_index_data.json"
}
},
"tuningConfig": {
"type": "index"
}
}
}

View File

@ -0,0 +1,71 @@
{
"type": "index_realtime",
"spec": {
"dataSchema": {
"dataSource": "wikipedia_index_test",
"metricsSpec": [
{
"type": "count",
"name": "count"
},
{
"type": "doubleSum",
"name": "added",
"fieldName": "added"
},
{
"type": "doubleSum",
"name": "deleted",
"fieldName": "deleted"
},
{
"type": "doubleSum",
"name": "delta",
"fieldName": "delta"
}
],
"granularitySpec": {
"segmentGranularity": "DAY",
"queryGranularity": "second"
},
"parser": {
"type" : "map",
"parseSpec": {
"timestampSpec": {
"column": "timestamp",
"format": "iso"
},
"dimensionsSpec" : {
"dimensions": [
"page", "language", "user", "unpatrolled", "newPage", "robot", "anonymous",
"namespace", "continent", "country", "region", "city"
]
}
}
}
},
"ioConfig": {
"type": "realtime",
"firehose": {
"type": "timed",
"shutoffTime": "#SHUTOFFTIME",
"delegate": {
"type": "receiver",
"serviceName": "eventReceiverServiceName",
"bufferSize": 100000
}
}
},
"tuningConfig": {
"type": "realtime",
"maxRowsInMemory": 1,
"intermediatePersistPeriod": "PT1M",
"windowPeriod": "PT1M",
"rejectionPolicy": {
"type": "none"
}
}
}
}

View File

@ -0,0 +1,62 @@
{
"type": "index",
"spec": {
"dataSchema": {
"dataSource": "wikipedia_index_test",
"metricsSpec": [
{
"type": "count",
"name": "count"
},
{
"type": "doubleSum",
"name": "added",
"fieldName": "added"
},
{
"type": "doubleSum",
"name": "deleted",
"fieldName": "deleted"
},
{
"type": "doubleSum",
"name": "delta",
"fieldName": "delta"
}
],
"granularitySpec": {
"segmentGranularity": "DAY",
"queryGranularity": "second",
"intervals" : [ "2013-08-31/2013-09-01" ]
},
"parser": {
"parseSpec": {
"format" : "json",
"timestampSpec": {
"column": "timestamp",
"format": "iso"
},
"dimensionsSpec": {
"dimensions": [
"page", "language", "user", "unpatrolled", "newPage", "robot", "anonymous",
"namespace", "continent", "country", "region", "city"
]
}
}
}
},
"ioConfig": {
"type": "index",
"firehose": {
"type": "ingestSegment",
"dataSource": "wikipedia_index_test",
"dimensions": ["user", "nonexist"],
"metrics": ["added", "added2"],
"interval": "2013-08-31/2013-09-01"
}
},
"tuningConfig": {
"type": "index"
}
}
}

View File

@ -0,0 +1,780 @@
[
{
"description": "timeseries, 2 aggs",
"query": {
"queryType": "timeseries",
"dataSource": "twitterstream",
"intervals": ["2013-01-01T00:00:00.000/2013-01-04T00:00:00.000"],
"granularity": "day",
"aggregations": [
{
"type": "doubleSum",
"name": "num_tweets",
"fieldName": "count"
},
{
"type": "doubleSum",
"name": "tweet_length",
"fieldName": "tweet_length"
}
],
"context": {
"useCache": "true",
"populateCache": "true",
"timeout": 60000
}
},
"expectedResults": [
{
"timestamp": "2013-01-01T00:00:00.000Z",
"result": {
"tweet_length": 2.40241323E8,
"num_tweets": 3754028.0
}
},
{
"timestamp": "2013-01-02T00:00:00.000Z",
"result": {
"tweet_length": 2.46397801E8,
"num_tweets": 3799466.0
}
},
{
"timestamp": "2013-01-03T00:00:00.000Z",
"result": {
"tweet_length": 2.31365019E8,
"num_tweets": 3552419.0
}
}
]
},
{
"description": "topN, 2 aggs, lexicographic",
"query": {
"queryType": "topN",
"dataSource": "twitterstream",
"intervals": ["2013-01-01T00:00:00.000/2013-01-04T00:00:00.000"],
"granularity": "day",
"aggregations": [
{
"type": "doubleSum",
"name": "num_tweets",
"fieldName": "count"
},
{
"type": "doubleSum",
"name": "tweet_length",
"fieldName": "tweet_length"
}
],
"postAggregations": [
{
"type": "arithmetic",
"name": "avg_tweet_len",
"fn": "/",
"fields": [
{
"type": "fieldAccess",
"name": "tweet_length",
"fieldName": "tweet_length"
},
{
"type": "fieldAccess",
"name": "num_tweets",
"fieldName": "num_tweets"
}
]
}
],
"dimension": "user_name",
"metric": {
"type": "lexicographic"
},
"threshold": 2,
"context": {
"useCache": "true",
"populateCache": "true",
"timeout": 60000
}
},
"expectedResults": [
{
"timestamp": "2013-01-01T00:00:00.000Z",
"result": [
{
"user_name": "000000000000087",
"tweet_length": 14.0,
"num_tweets": 1.0,
"avg_tweet_len": 14.0
},
{
"user_name": "0000000000mghi",
"tweet_length": 291.0,
"num_tweets": 4.0,
"avg_tweet_len": 72.75
}
]
},
{
"timestamp": "2013-01-02T00:00:00.000Z",
"result": [
{
"user_name": "000000000037",
"tweet_length": 13.0,
"num_tweets": 1.0,
"avg_tweet_len": 13.0
},
{
"user_name": "0000000000mghi",
"tweet_length": 21.0,
"num_tweets": 1.0,
"avg_tweet_len": 21.0
}
]
},
{
"timestamp": "2013-01-03T00:00:00.000Z",
"result": [
{
"user_name": "000000007",
"tweet_length": 37.0,
"num_tweets": 1.0,
"avg_tweet_len": 37.0
},
{
"user_name": "00000000b",
"tweet_length": 119.0,
"num_tweets": 1.0,
"avg_tweet_len": 119.0
}
]
}
]
},
{
"description": "topN, 2 aggs",
"query": {
"queryType": "topN",
"dataSource": "twitterstream",
"intervals": ["2013-01-01T00:00:00.000/2013-01-04T00:00:00.000"],
"granularity": "day",
"aggregations": [
{
"type": "doubleSum",
"name": "num_tweets",
"fieldName": "count"
},
{
"type": "doubleSum",
"name": "tweet_length",
"fieldName": "tweet_length"
}
],
"postAggregations": [
{
"type": "arithmetic",
"name": "avg_tweet_len",
"fn": "/",
"fields": [
{
"type": "fieldAccess",
"name": "tweet_length",
"fieldName": "tweet_length"
},
{
"type": "fieldAccess",
"name": "num_tweets",
"fieldName": "num_tweets"
}
]
}
],
"dimension": "user_name",
"metric": {
"type": "numeric",
"metric": "num_tweets"
},
"threshold": 2,
"context": {
"useCache": "true",
"populateCache": "true",
"timeout": 60000
}
},
"expectedResults": [
{
"timestamp": "2013-01-01T00:00:00.000Z",
"result": [
{
"user_name": "Favstar_Bot",
"tweet_length": 2002.0,
"num_tweets": 33.0,
"avg_tweet_len": 60.666666666666664
},
{
"user_name": "SportsAB",
"tweet_length": 1114.0,
"num_tweets": 26.0,
"avg_tweet_len": 42.84615384615385
}
]
},
{
"timestamp": "2013-01-02T00:00:00.000Z",
"result": [
{
"user_name": "Favstar_Bot",
"tweet_length": 2185.0,
"num_tweets": 36.0,
"avg_tweet_len": 60.69444444444444
},
{
"user_name": "SportsAB",
"tweet_length": 1148.0,
"num_tweets": 23.0,
"avg_tweet_len": 49.91304347826087
}
]
},
{
"timestamp": "2013-01-03T00:00:00.000Z",
"result": [
{
"user_name": "SportsAB",
"tweet_length": 882.0,
"num_tweets": 22.0,
"avg_tweet_len": 40.09090909090909
},
{
"user_name": "furin0620",
"tweet_length": 867.0,
"num_tweets": 21.0,
"avg_tweet_len": 41.285714285714285
}
]
}
]
},
{
"description": "topN, 2 aggs, filtered",
"query": {
"queryType": "topN",
"dataSource": "twitterstream",
"intervals": ["2013-01-01T00:00:00.000/2013-01-04T00:00:00.000"],
"granularity": "day",
"filter": {
"type": "or",
"fields": [
{
"type": "selector",
"dimension": "user_name",
"value": "Favstar_Bot"
},
{
"type": "selector",
"dimension": "user_name",
"value": "SportsAB"
},
{
"type": "selector",
"dimension": "user_name",
"value": "furin0620"
}
]
},
"aggregations": [
{
"type": "doubleSum",
"name": "num_tweets",
"fieldName": "count"
},
{
"type": "doubleSum",
"name": "tweet_length",
"fieldName": "tweet_length"
}
],
"postAggregations": [
{
"type": "arithmetic",
"name": "avg_tweet_len",
"fn": "/",
"fields": [
{
"type": "fieldAccess",
"name": "tweet_length",
"fieldName": "tweet_length"
},
{
"type": "fieldAccess",
"name": "num_tweets",
"fieldName": "num_tweets"
}
]
}
],
"dimension": "user_name",
"metric": {
"type": "numeric",
"metric": "num_tweets"
},
"threshold": 2,
"context": {
"useCache": "true",
"populateCache": "true",
"timeout": 60000
}
},
"expectedResults": [
{
"timestamp": "2013-01-01T00:00:00.000Z",
"result": [
{
"user_name": "Favstar_Bot",
"tweet_length": 2002.0,
"num_tweets": 33.0,
"avg_tweet_len": 60.666666666666664
},
{
"user_name": "SportsAB",
"tweet_length": 1114.0,
"num_tweets": 26.0,
"avg_tweet_len": 42.84615384615385
}
]
},
{
"timestamp": "2013-01-02T00:00:00.000Z",
"result": [
{
"user_name": "Favstar_Bot",
"tweet_length": 2185.0,
"num_tweets": 36.0,
"avg_tweet_len": 60.69444444444444
},
{
"user_name": "SportsAB",
"tweet_length": 1148.0,
"num_tweets": 23.0,
"avg_tweet_len": 49.91304347826087
}
]
},
{
"timestamp": "2013-01-03T00:00:00.000Z",
"result": [
{
"user_name": "SportsAB",
"tweet_length": 882.0,
"num_tweets": 22.0,
"avg_tweet_len": 40.09090909090909
},
{
"user_name": "furin0620",
"tweet_length": 867.0,
"num_tweets": 21.0,
"avg_tweet_len": 41.285714285714285
}
]
}
]
},
{
"description": "groupBy",
"query": {
"queryType": "groupBy",
"dataSource": "twitterstream",
"intervals": ["2013-01-01T00:00:00.000/2013-01-04T00:00:00.000"],
"granularity": "day",
"aggregations": [
{
"type": "doubleSum",
"name": "num_tweets",
"fieldName": "count"
},
{
"type": "doubleSum",
"name": "tweet_length",
"fieldName": "tweet_length"
}
],
"dimensions": ["has_links"]
},
"expectedResults": [
{
"version": "v1",
"timestamp": "2013-01-01T00:00:00.000Z",
"event": {
"has_links": "No",
"tweet_length": 2.08803904E8,
"num_tweets": 3377791.0
}
},
{
"version": "v1",
"timestamp": "2013-01-01T00:00:00.000Z",
"event": {
"has_links": "Yes",
"tweet_length": 3.143742E7,
"num_tweets": 376237.0
}
},
{
"version": "v1",
"timestamp": "2013-01-02T00:00:00.000Z",
"event": {
"has_links": "No",
"tweet_length": 2.10402688E8,
"num_tweets": 3375243.0
}
},
{
"version": "v1",
"timestamp": "2013-01-02T00:00:00.000Z",
"event": {
"has_links": "Yes",
"tweet_length": 3.599512E7,
"num_tweets": 424223.0
}
},
{
"version": "v1",
"timestamp": "2013-01-03T00:00:00.000Z",
"event": {
"has_links": "No",
"tweet_length": 1.96451456E8,
"num_tweets": 3144985.0
}
},
{
"version": "v1",
"timestamp": "2013-01-03T00:00:00.000Z",
"event": {
"has_links": "Yes",
"tweet_length": 3.4913568E7,
"num_tweets": 407434.0
}
}
]
},
{
"query": {
"queryType": "search",
"intervals": ["2013-01-01T00:00:00.000/2013-01-04T00:00:00.000"],
"dataSource": "twitterstream",
"granularity": "all",
"searchDimensions": ["user_name"],
"sort": {
"type": "lexicographic"
},
"query": {
"type": "insensitive_contains",
"value": "Sports"
},
"limit": 3,
"context": {
"useCache": "true",
"populateCache": "true",
"timeout": 60000
}
},
"expectedResults": [
{
"timestamp": "2013-01-01T00:00:00.000Z",
"result": [
{
"dimension": "user_name",
"value": "1011Sports"
},
{
"dimension": "user_name",
"value": "11AliveSports"
},
{
"dimension": "user_name",
"value": "1World_Sports"
}
]
}
]
},
{
"description": "groupByArbitraryInterval",
"query": {
"queryType": "groupBy",
"dataSource": "twitterstream",
"intervals": ["2013-01-01T15:10:10.090/2013-01-03T19:30:01.090"],
"granularity": "day",
"aggregations": [
{
"type": "doubleSum",
"name": "num_tweets",
"fieldName": "count"
},
{
"type": "doubleSum",
"name": "tweet_length",
"fieldName": "tweet_length"
}
],
"dimensions": ["has_links"]
},
"expectedResults": [
{
"version": "v1",
"timestamp": "2013-01-01T00:00:00.000Z",
"event": {
"has_links": "No",
"tweet_length": 7.4820448E7,
"num_tweets": 1170229.0
}
},
{
"version": "v1",
"timestamp": "2013-01-01T00:00:00.000Z",
"event": {
"has_links": "Yes",
"tweet_length": 1.149719E7,
"num_tweets": 136582.0
}
},
{
"version": "v1",
"timestamp": "2013-01-02T00:00:00.000Z",
"event": {
"has_links": "No",
"tweet_length": 2.10402688E8,
"num_tweets": 3375243.0
}
},
{
"version": "v1",
"timestamp": "2013-01-02T00:00:00.000Z",
"event": {
"has_links": "Yes",
"tweet_length": 3.599512E7,
"num_tweets": 424223.0
}
},
{
"version": "v1",
"timestamp": "2013-01-03T00:00:00.000Z",
"event": {
"has_links": "No",
"tweet_length": 1.59141088E8,
"num_tweets": 2567986.0
}
},
{
"version": "v1",
"timestamp": "2013-01-03T00:00:00.000Z",
"event": {
"has_links": "Yes",
"tweet_length": 2.8345444E7,
"num_tweets": 328917.0
}
}
]
},
{
"description": "segmentMetadata",
"query": {
"queryType": "segmentMetadata",
"dataSource": "twitterstream",
"intervals": ["2013-01-01T00:00:00.000/2013-01-04T00:00:00.000"],
"toInclude": {
"type": "list",
"columns": ["has_links", "has_links"]
}
},
"expectedResults": [
{
"id": "twitterstream_2013-01-01T00:00:00.000Z_2013-01-02T00:00:00.000Z_2013-01-02T04:13:41.980Z_v9",
"intervals": ["2013-01-01T00:00:00.000Z/2013-01-02T00:00:00.000Z"],
"columns": {
"has_links": {
"type": "STRING",
"size": 7773438,
"cardinality": 2,
"errorMessage": null
}
},
"size": 747056474
},
{
"id": "twitterstream_2013-01-02T00:00:00.000Z_2013-01-03T00:00:00.000Z_2013-01-03T03:44:58.791Z_v9",
"intervals": ["2013-01-02T00:00:00.000Z/2013-01-03T00:00:00.000Z"],
"columns": {
"has_links": {
"type": "STRING",
"size": 7901000,
"cardinality": 2,
"errorMessage": null
}
},
"size": 755796690
},
{
"id": "twitterstream_2013-01-03T00:00:00.000Z_2013-01-04T00:00:00.000Z_2013-01-04T04:09:13.590Z_v9",
"intervals": ["2013-01-03T00:00:00.000Z/2013-01-04T00:00:00.000Z"],
"columns": {
"has_links": {
"type": "STRING",
"size": 7405654,
"cardinality": 2,
"errorMessage": null
}
},
"size": 706893542
}
]
},
{
"description": "topN, 2 aggs, topN over dependent postAgg",
"query": {
"queryType": "topN",
"dataSource": "twitterstream",
"intervals": ["2013-01-01T00:00:00.000/2013-01-04T00:00:00.000"],
"granularity": "day",
"aggregations": [
{
"type": "doubleSum",
"name": "num_tweets",
"fieldName": "count"
},
{
"type": "doubleSum",
"name": "tweet_length",
"fieldName": "tweet_length"
}
],
"postAggregations": [
{
"type": "arithmetic",
"name": "avg_tweet_len",
"fn": "/",
"fields": [
{
"type": "fieldAccess",
"name": "tweet_length",
"fieldName": "tweet_length"
},
{
"type": "fieldAccess",
"name": "num_tweets",
"fieldName": "num_tweets"
}
]
},
{
"type": "arithmetic",
"name": "avg_tweet_len_half",
"fn": "/",
"fields": [
{
"type": "fieldAccess",
"name": "avg_tweet_len",
"fieldName": "avg_tweet_len"
},
{
"type": "constant",
"value": "2"
}
]
},
{
"type": "arithmetic",
"name": "avg_tweet_len_doubled",
"fn": "*",
"fields": [
{
"type": "fieldAccess",
"name": "avg_tweet_len",
"fieldName": "avg_tweet_len"
},
{
"type": "constant",
"value": "2"
}
]
}
],
"dimension": "user_name",
"metric": {
"type": "numeric",
"metric": "avg_tweet_len_doubled"
},
"threshold": 2,
"context": {
"useCache": "true",
"populateCache": "true",
"timeout": 60000
}
},
"expectedResults": [
{
"timestamp": "2013-01-01T00:00:00.000Z",
"result": [
{
"user_name": "___soMALIa___",
"tweet_length": 539.0,
"avg_tweet_len_half": 269.5,
"avg_tweet_len_doubled": 1078.0,
"num_tweets": 1.0,
"avg_tweet_len": 539.0
},
{
"user_name": "SophiiiaSlr",
"tweet_length": 530.0,
"avg_tweet_len_half": 265.0,
"avg_tweet_len_doubled": 1060.0,
"num_tweets": 1.0,
"avg_tweet_len": 530.0
}
]
},
{
"timestamp": "2013-01-02T00:00:00.000Z",
"result": [
{
"user_name": "FallenReckless",
"tweet_length": 518.0,
"avg_tweet_len_half": 259.0,
"avg_tweet_len_doubled": 1036.0,
"num_tweets": 1.0,
"avg_tweet_len": 518.0
},
{
"user_name": "SigaMike",
"tweet_length": 514.0,
"avg_tweet_len_half": 257.0,
"avg_tweet_len_doubled": 1028.0,
"num_tweets": 1.0,
"avg_tweet_len": 514.0
}
]
},
{
"timestamp": "2013-01-03T00:00:00.000Z",
"result": [
{
"user_name": "Alejo_InReverse",
"tweet_length": 560.0,
"avg_tweet_len_half": 280.0,
"avg_tweet_len_doubled": 1120.0,
"num_tweets": 1.0,
"avg_tweet_len": 560.0
},
{
"user_name": "GavLeftHome",
"tweet_length": 506.0,
"avg_tweet_len_half": 253.0,
"avg_tweet_len_doubled": 1012.0,
"num_tweets": 1.0,
"avg_tweet_len": 506.0
}
]
}
]
}
]

File diff suppressed because it is too large Load Diff

View File

@ -0,0 +1,10 @@
<!DOCTYPE suite SYSTEM "http://testng.org/testng-1.0.dtd" >
<suite name="IntegrationTestSuite">
<test name="AllTests">
<packages>
<package name="io.druid.tests.*"/>
</packages>
</test>
</suite>

View File

@ -0,0 +1,5 @@
for node in druid-historical druid-coordinator druid-overlord druid-router druid-broker druid-middlemanager druid-zookeeper druid-metadata-storage;
do
docker stop $node
docker rm $node
done

View File

@ -55,6 +55,7 @@
<module>processing</module>
<module>server</module>
<module>services</module>
<module>integration-tests</module>
<!-- Non-default modules -->
<module>extensions/cassandra-storage</module>
<module>extensions/hdfs-storage</module>
@ -509,6 +510,11 @@
</exclusion>
</exclusions>
</dependency>
<dependency>
<groupId>org.testng</groupId>
<artifactId>testng</artifactId>
<version>6.8.7</version>
</dependency>
</dependencies>
</dependencyManagement>