diff --git a/docs/content/Tutorial:-A-First-Look-at-Druid.md b/docs/content/Tutorial:-A-First-Look-at-Druid.md index 39d908dbbd0..21ee6a6253e 100644 --- a/docs/content/Tutorial:-A-First-Look-at-Druid.md +++ b/docs/content/Tutorial:-A-First-Look-at-Druid.md @@ -3,7 +3,7 @@ layout: doc_page --- # Tutorial: A First Look at Druid -Greetings! This tutorial will help clarify some core Druid concepts. We will use a realtime dataset and issue some basic Druid queries. If you are ready to explore Druid, and learn a thing or two, read on! +Greetings! This tutorial will help clarify some core Druid concepts. We will use a real-time dataset and issue some basic Druid queries. If you are ready to explore Druid, and learn a thing or two, read on! About the data -------------- diff --git a/examples/bin/examples/indexing/wikipedia.spec b/examples/bin/examples/indexing/wikipedia.spec index 6d7f1499b25..5f626a42503 100644 --- a/examples/bin/examples/indexing/wikipedia.spec +++ b/examples/bin/examples/indexing/wikipedia.spec @@ -1,60 +1,73 @@ [ { - "schema": { - "dataSource": "wikipedia", - "aggregators" : [{ - "type" : "count", - "name" : "count" - }, { - "type" : "doubleSum", - "name" : "added", - "fieldName" : "added" - }, { - "type" : "doubleSum", - "name" : "deleted", - "fieldName" : "deleted" - }, { - "type" : "doubleSum", - "name" : "delta", - "fieldName" : "delta" - }], - "indexGranularity": "none" - }, - "config": { - "maxRowsInMemory": 500000, - "intermediatePersistPeriod": "PT10m" - }, - "firehose": { - "type": "kafka-0.7.2", - "consumerProps": { - "zk.connect": "localhost:2181", - "zk.connectiontimeout.ms": "15000", - "zk.sessiontimeout.ms": "15000", - "zk.synctime.ms": "5000", - "groupid": "druid-example", - "fetch.size": "1048586", - "autooffset.reset": "largest", - "autocommit.enable": "false" - }, - "feed": "wikipedia", - "parser": { - "timestampSpec": { - "column": "timestamp" - }, - "data": { - "format": "json", - "dimensions" : ["page","language","user","unpatrolled","newPage","robot","anonymous","namespace","continent","country","region","city"] + "dataSchema" : { + "dataSource" : "wikipedia", + "parser" : { + "type" : "string", + "parseSpec" : { + "format" : "json", + "timestampSpec" : { + "column" : "timestamp", + "format" : "auto" + }, + "dimensionsSpec" : { + "dimensions": ["page","language","user","unpatrolled","newPage","robot","anonymous","namespace","continent","country","region","city"], + "dimensionExclusions" : [], + "spatialDimensions" : [] + } } + }, + "metricsSpec" : [{ + "type" : "count", + "name" : "count" + }, { + "type" : "doubleSum", + "name" : "added", + "fieldName" : "added" + }, { + "type" : "doubleSum", + "name" : "deleted", + "fieldName" : "deleted" + }, { + "type" : "doubleSum", + "name" : "delta", + "fieldName" : "delta" + }], + "granularitySpec" : { + "type" : "uniform", + "segmentGranularity" : "DAY", + "queryGranularity" : "NONE" } }, - "plumber": { - "type": "realtime", + "ioConfig" : { + "type" : "realtime", + "firehose": { + "type": "kafka-0.7.2", + "consumerProps": { + "zk.connect": "localhost:2181", + "zk.connectiontimeout.ms": "15000", + "zk.sessiontimeout.ms": "15000", + "zk.synctime.ms": "5000", + "groupid": "druid-example", + "fetch.size": "1048586", + "autooffset.reset": "largest", + "autocommit.enable": "false" + }, + "feed": "wikipedia" + }, + "plumber": { + "type": "realtime" + } + }, + "tuningConfig": { + "type" : "realtime", + "maxRowsInMemory": 500000, + "intermediatePersistPeriod": "PT10m", "windowPeriod": "PT10m", - "segmentGranularity": "hour", "basePersistDirectory": "\/tmp\/realtime\/basePersist", "rejectionPolicy": { - "type": "test" + "type": "messageTime" } } } -] \ No newline at end of file +] diff --git a/examples/bin/examples/indexing/wikipedia_hadoop_config.json b/examples/bin/examples/indexing/wikipedia_hadoop_config.json index cafc4e8428b..964016405c4 100644 --- a/examples/bin/examples/indexing/wikipedia_hadoop_config.json +++ b/examples/bin/examples/indexing/wikipedia_hadoop_config.json @@ -1,51 +1,82 @@ { - "dataSource": "wikipedia", - "timestampSpec" : { - "column": "timestamp", - "format": "iso" + "dataSchema": { + "dataSource": "wikipedia", + "parser": { + "type": "string", + "parseSpec": { + "format": "json", + "timestampSpec": { + "column": "timestamp", + "format": "auto" + }, + "dimensionsSpec": { + "dimensions": [ + "page", + "language", + "user", + "unpatrolled", + "newPage", + "robot", + "anonymous", + "namespace", + "continent", + "country", + "region", + "city" + ], + "dimensionExclusions": [], + "spatialDimensions": [] + } + } + }, + "metricsSpec": [ + { + "type": "count", + "name": "count" + }, + { + "type": "doubleSum", + "name": "added", + "fieldName": "added" + }, + { + "type": "doubleSum", + "name": "deleted", + "fieldName": "deleted" + }, + { + "type": "doubleSum", + "name": "delta", + "fieldName": "delta" + } + ], + "granularitySpec": { + "type": "uniform", + "segmentGranularity": "DAY", + "queryGranularity": "NONE", + "intervals": ["2013-08-31/2013-09-01"] + } }, - "dataSpec": { - "format": "json", - "dimensions" : ["page","language","user","unpatrolled","newPage","robot","anonymous","namespace","continent","country","region","city"] + "ioConfig": { + "type": "hadoop", + "inputSpec": { + "type": "static", + "paths": "/myPath/druid-services-0.6.160/examples/indexing/wikipedia_data.json" + }, + "metadataUpdateSpec": { + "type": "db", + "connectURI": "jdbc:mysql:\/\/localhost:3306\/druid", + "user": "druid", + "password": "diurd", + "segmentTable": "druid_segments" + }, + "segmentOutputPath": "\/tmp\/segments" }, - "granularitySpec" : { - "type" : "uniform", - "gran" : "DAY", - "intervals" : [ "2013-08-31/2013-09-01" ] - }, - "pathSpec": { - "type": "static", - "paths": "examples/indexing/wikipedia_data.json" - }, - "rollupSpec": { - "aggs": [{ - "type" : "count", - "name" : "count" - }, { - "type" : "doubleSum", - "name" : "added", - "fieldName" : "added" - }, { - "type" : "doubleSum", - "name" : "deleted", - "fieldName" : "deleted" - }, { - "type" : "doubleSum", - "name" : "delta", - "fieldName" : "delta" - }], - "rollupGranularity": "none" - }, - "workingPath": "\/tmp\/working_path", - "segmentOutputPath": "\/tmp\/segments", - "partitionsSpec": { - "targetPartitionSize": 5000000 - }, - "metadataUpdateSpec": { - "type": "db", - "connectURI": "jdbc:mysql:\/\/localhost:3306\/druid", - "user": "druid", - "password": "diurd", - "segmentTable": "druid_segments" + "tuningConfig": { + "type": "hadoop", + "workingPath": "\/tmp\/working_path", + "partitionsSpec": { + "targetPartitionSize": 5000000 + } } } \ No newline at end of file diff --git a/examples/bin/examples/indexing/wikipedia_index_hadoop_task.json b/examples/bin/examples/indexing/wikipedia_index_hadoop_task.json index 46a4a689758..3374f00ada3 100644 --- a/examples/bin/examples/indexing/wikipedia_index_hadoop_task.json +++ b/examples/bin/examples/indexing/wikipedia_index_hadoop_task.json @@ -1,43 +1,76 @@ { - "type" : "index_hadoop", - "config": { - "dataSource" : "wikipedia", - "timestampSpec" : { - "column": "timestamp", - "format": "auto" + "type": "index_hadoop", + "spec": { + "dataSchema": { + "dataSource": "wikipedia", + "parser": { + "type": "string", + "parseSpec": { + "format": "json", + "timestampSpec": { + "column": "timestamp", + "format": "auto" + }, + "dimensionsSpec": { + "dimensions": [ + "page", + "language", + "user", + "unpatrolled", + "newPage", + "robot", + "anonymous", + "namespace", + "continent", + "country", + "region", + "city" + ], + "dimensionExclusions": [], + "spatialDimensions": [] + } + } + }, + "metricsSpec": [ + { + "type": "count", + "name": "count" + }, + { + "type": "doubleSum", + "name": "added", + "fieldName": "added" + }, + { + "type": "doubleSum", + "name": "deleted", + "fieldName": "deleted" + }, + { + "type": "doubleSum", + "name": "delta", + "fieldName": "delta" + } + ], + "granularitySpec": { + "type": "uniform", + "segmentGranularity": "DAY", + "queryGranularity": "NONE", + "intervals": ["2013-08-31/2013-09-01"] + } }, - "dataSpec" : { - "format" : "json", - "dimensions" : ["page","language","user","unpatrolled","newPage","robot","anonymous","namespace","continent","country","region","city"] + "ioConfig": { + "type": "hadoop", + "inputSpec": { + "type": "static", + "paths": "/myPath/druid-services-0.6.160/examples/indexing/wikipedia_data.json" + } }, - "granularitySpec" : { - "type" : "uniform", - "gran" : "DAY", - "intervals" : [ "2013-08-31/2013-09-01" ] - }, - "pathSpec" : { - "type" : "static", - "paths" : "examples/indexing/wikipedia_data.json" - }, - "targetPartitionSize" : 5000000, - "rollupSpec" : { - "aggs": [{ - "type" : "count", - "name" : "count" - }, { - "type" : "doubleSum", - "name" : "added", - "fieldName" : "added" - }, { - "type" : "doubleSum", - "name" : "deleted", - "fieldName" : "deleted" - }, { - "type" : "doubleSum", - "name" : "delta", - "fieldName" : "delta" - }], - "rollupGranularity" : "none" + "tuningConfig": { + "type": "hadoop", + "partitionsSpec": { + "targetPartitionSize": 5000000 + } } } } \ No newline at end of file diff --git a/examples/bin/examples/indexing/wikipedia_index_task.json b/examples/bin/examples/indexing/wikipedia_index_task.json index 630de0e1405..3b64bbde635 100644 --- a/examples/bin/examples/indexing/wikipedia_index_task.json +++ b/examples/bin/examples/indexing/wikipedia_index_task.json @@ -1,39 +1,76 @@ { - "type" : "index", - "dataSource" : "wikipedia", - "granularitySpec" : { - "type" : "uniform", - "gran" : "DAY", - "intervals" : [ "2013-08-31/2013-09-01" ] - }, - "aggregators" : [{ - "type" : "count", - "name" : "count" - }, { - "type" : "doubleSum", - "name" : "added", - "fieldName" : "added" - }, { - "type" : "doubleSum", - "name" : "deleted", - "fieldName" : "deleted" - }, { - "type" : "doubleSum", - "name" : "delta", - "fieldName" : "delta" - }], - "firehose" : { - "type" : "local", - "baseDir" : "examples/indexing/", - "filter" : "wikipedia_data.json", - "parser" : { - "timestampSpec" : { - "column" : "timestamp" + "type": "index", + "spec": { + "dataSchema": { + "dataSource": "wikipedia", + "parser": { + "type": "string", + "parseSpec": { + "format": "json", + "timestampSpec": { + "column": "timestamp", + "format": "auto" + }, + "dimensionsSpec": { + "dimensions": [ + "page", + "language", + "user", + "unpatrolled", + "newPage", + "robot", + "anonymous", + "namespace", + "continent", + "country", + "region", + "city" + ], + "dimensionExclusions": [], + "spatialDimensions": [] + } + } }, - "data" : { - "format" : "json", - "dimensions" : ["page","language","user","unpatrolled","newPage","robot","anonymous","namespace","continent","country","region","city"] + "metricsSpec": [ + { + "type": "count", + "name": "count" + }, + { + "type": "doubleSum", + "name": "added", + "fieldName": "added" + }, + { + "type": "doubleSum", + "name": "deleted", + "fieldName": "deleted" + }, + { + "type": "doubleSum", + "name": "delta", + "fieldName": "delta" + } + ], + "granularitySpec": { + "type": "uniform", + "segmentGranularity": "DAY", + "queryGranularity": "NONE", + "intervals": ["2013-08-31/2013-09-01"] } + }, + "ioConfig": { + "type": "index", + "firehose": { + "type": "local", + "baseDir": "/MyPath/druid-services-0.6.160/examples/indexing/", + "filter": "wikipedia_data.json" + } + }, + "tuningConfig": { + "type": "index", + "targetPartitionSize": 0, + "rowFlushBoundary": 0 } } } \ No newline at end of file diff --git a/examples/bin/examples/indexing/wikipedia_kafka_realtime_task.json b/examples/bin/examples/indexing/wikipedia_kafka_realtime_task.json index ec08392abf4..e4c76b2248d 100644 --- a/examples/bin/examples/indexing/wikipedia_kafka_realtime_task.json +++ b/examples/bin/examples/indexing/wikipedia_kafka_realtime_task.json @@ -1,73 +1,92 @@ { - "type": "index_realtime", - "schema": { - "dataSource": "wikipedia", - "aggregators": [ - { - "type": "count", - "name": "count" + "type" : "index_realtime", + "spec" : { + "dataSchema": { + "dataSource": "wikipedia", + "parser": { + "type": "string", + "parseSpec": { + "format": "json", + "timestampSpec": { + "column": "timestamp", + "format": "auto" + }, + "dimensionsSpec": { + "dimensions": [ + "page", + "language", + "user", + "unpatrolled", + "newPage", + "robot", + "anonymous", + "namespace", + "continent", + "country", + "region", + "city" + ], + "dimensionExclusions": [], + "spatialDimensions": [] + } + } }, - { - "type": "doubleSum", - "name": "added", - "fieldName": "added" - }, - { - "type": "doubleSum", - "name": "deleted", - "fieldName": "deleted" - }, - { - "type": "doubleSum", - "name": "delta", - "fieldName": "delta" + "metricsSpec": [ + { + "type": "count", + "name": "count" + }, + { + "type": "doubleSum", + "name": "added", + "fieldName": "added" + }, + { + "type": "doubleSum", + "name": "deleted", + "fieldName": "deleted" + }, + { + "type": "doubleSum", + "name": "delta", + "fieldName": "delta" + } + ], + "granularitySpec": { + "type": "uniform", + "segmentGranularity": "DAY", + "queryGranularity": "NONE" } - ], - "indexGranularity": "none" - }, - "fireDepartmentConfig": { - "maxRowsInMemory": 500000, - "intermediatePersistPeriod": "PT10m" - }, - "firehose": { - "type": "kafka-0.7.2", - "consumerProps": { - "zk.connect": "localhost:2181", - "zk.connectiontimeout.ms": "15000", - "zk.sessiontimeout.ms": "15000", - "zk.synctime.ms": "5000", - "groupid": "druid-example", - "fetch.size": "1048586", - "autooffset.reset": "largest", - "autocommit.enable": "false" }, - "feed": "wikipedia", - "parser": { - "timestampSpec": { - "column": "timestamp" + "ioConfig": { + "type": "realtime", + "firehose": { + "type": "kafka-0.7.2", + "consumerProps": { + "zk.connect": "localhost:2181", + "zk.connectiontimeout.ms": "15000", + "zk.sessiontimeout.ms": "15000", + "zk.synctime.ms": "5000", + "groupid": "druid-example", + "fetch.size": "1048586", + "autooffset.reset": "largest", + "autocommit.enable": "false" + }, + "feed": "wikipedia" }, - "data": { - "format": "json", - "dimensions": [ - "page", - "language", - "user", - "unpatrolled", - "newPage", - "robot", - "anonymous", - "namespace", - "continent", - "country", - "region", - "city" - ] + "plumber": { + "type": "realtime" + } + }, + "tuningConfig": { + "type": "realtime", + "maxRowsInMemory": 500000, + "intermediatePersistPeriod": "PT10m", + "windowPeriod": "PT10m", + "basePersistDirectory": "\/tmp\/realtime\/basePersist", + "rejectionPolicy": { + "type": "serverTime" } } - }, - "windowPeriod": "PT10m", - "segmentGranularity": "hour", - "rejectionPolicy": { - "type": "test" } } \ No newline at end of file diff --git a/examples/bin/examples/indexing/wikipedia_realtime_task.json b/examples/bin/examples/indexing/wikipedia_realtime_task.json index 3365c50368c..7110fa4de0d 100644 --- a/examples/bin/examples/indexing/wikipedia_realtime_task.json +++ b/examples/bin/examples/indexing/wikipedia_realtime_task.json @@ -1,71 +1,113 @@ { "type": "index_realtime", - "schema": { - "dataSource": "wikipedia", - "aggregators": [ - { - "type": "count", - "name": "count" - }, - { - "type": "doubleSum", - "name": "added", - "fieldName": "added" - }, - { - "type": "doubleSum", - "name": "deleted", - "fieldName": "deleted" - }, - { - "type": "doubleSum", - "name": "delta", - "fieldName": "delta" - } - ], - "indexGranularity": "none" - }, - "fireDepartmentConfig": { - "maxRowsInMemory": 500000, - "intermediatePersistPeriod": "PT10m" - }, - "firehose": { - "type": "irc", - "nick": "wiki1234567890", - "host": "irc.wikimedia.org", - "channels": [ - "#en.wikipedia", - "#fr.wikipedia", - "#de.wikipedia", - "#ja.wikipedia" - ], - "decoder": { - "type": "wikipedia", - "namespaces": { - "#en.wikipedia": { - "_empty_": "main", - "Category": "category", - "$1 talk": "project talk", - "Template talk": "template talk", - "Help talk": "help talk", - "Media": "media", - "MediaWiki talk": "mediawiki talk", - "File talk": "file talk", - "MediaWiki": "mediawiki", - "User": "user", - "File": "file", - "User talk": "user talk", - "Template": "template", - "Help": "help", - "Special": "special", - "Talk": "talk", - "Category talk": "category talk" + "spec": { + "dataSchema": { + "dataSource": "wikipedia", + "parser": { + "type": "irc", + "parseSpec": { + "format": "json", + "timestampSpec": { + "column": "timestamp", + "format": "iso" + }, + "dimensionsSpec": { + "dimensions": [ + "page", + "language", + "user", + "unpatrolled", + "newPage", + "robot", + "anonymous", + "namespace", + "continent", + "country", + "region", + "city" + ], + "dimensionExclusions": [], + "spatialDimensions": [] + } + }, + "decoder": { + "type": "wikipedia", + "namespaces": { + "#en.wikipedia": { + "_empty_": "main", + "Category": "category", + "$1 talk": "project talk", + "Template talk": "template talk", + "Help talk": "help talk", + "Media": "media", + "MediaWiki talk": "mediawiki talk", + "File talk": "file talk", + "MediaWiki": "mediawiki", + "User": "user", + "File": "file", + "User talk": "user talk", + "Template": "template", + "Help": "help", + "Special": "special", + "Talk": "talk", + "Category talk": "category talk" + } + } } + }, + "metricsSpec": [ + { + "type": "count", + "name": "count" + }, + { + "type": "doubleSum", + "name": "added", + "fieldName": "added" + }, + { + "type": "doubleSum", + "name": "deleted", + "fieldName": "deleted" + }, + { + "type": "doubleSum", + "name": "delta", + "fieldName": "delta" + } + ], + "granularitySpec": { + "type": "uniform", + "segmentGranularity": "DAY", + "queryGranularity": "NONE" } }, - "timeDimension": "timestamp", - "timeFormat": "iso" - }, - "windowPeriod": "PT10m", - "segmentGranularity": "hour" + "ioConfig": { + "type": "realtime", + "firehose": { + "type": "irc", + "nick": "wiki1234567890", + "host": "irc.wikimedia.org", + "channels": [ + "#en.wikipedia", + "#fr.wikipedia", + "#de.wikipedia", + "#ja.wikipedia" + ] + }, + "plumber": { + "type": "realtime" + } + }, + "tuningConfig": { + "type": "realtime", + "maxRowsInMemory": 500000, + "intermediatePersistPeriod": "PT10m", + "windowPeriod": "PT10m", + "basePersistDirectory": "\/tmp\/realtime\/basePersist", + "rejectionPolicy": { + "type": "serverTime" + } + } + } } \ No newline at end of file diff --git a/examples/bin/examples/rabbitmq/query.body b/examples/bin/examples/rabbitmq/query.body deleted file mode 100644 index 43623cfa733..00000000000 --- a/examples/bin/examples/rabbitmq/query.body +++ /dev/null @@ -1,12 +0,0 @@ -{ - "queryType": "groupBy", - "dataSource": "rabbitmqtest", - "granularity": "all", - "dimensions": [], - "aggregations": [ - { "type": "count", "name": "rows" }, - {"type": "longSum", "name": "imps", "fieldName": "impressions"}, - {"type": "doubleSum", "name": "wp", "fieldName": "wp"} - ], - "intervals": ["2010-01-01T00:00/2020-01-01T00:00"] -} diff --git a/examples/bin/examples/rabbitmq/rabbitmq_realtime.spec b/examples/bin/examples/rabbitmq/rabbitmq_realtime.spec deleted file mode 100644 index c9f0f192b18..00000000000 --- a/examples/bin/examples/rabbitmq/rabbitmq_realtime.spec +++ /dev/null @@ -1,48 +0,0 @@ -[{ - "schema" : { - "dataSource":"rabbitmqtest", - "aggregators":[ - {"type":"count", "name":"impressions"}, - {"type":"doubleSum","name":"wp","fieldName":"wp"} - ], - "indexGranularity":"minute", - "shardSpec" : { "type": "none" } - }, - "config" : { - "maxRowsInMemory" : 500000, - "intermediatePersistPeriod" : "PT1m" - }, - "firehose" : { - "type" : "rabbitmq", - "connection" : { - "host": "localhost", - "username": "test-dude", - "password": "word-dude", - "virtualHost": "test-vhost" - }, - "config" : { - "exchange": "test-exchange", - "queue" : "druidtest", - "routingKey": "#", - "durable": "true", - "exclusive": "false", - "autoDelete": "false", - - "maxRetries": "10", - "retryIntervalSeconds": "1", - "maxDurationSeconds": "300" - }, - "parser" : { - "timestampSpec" : { "column" : "utcdt", "format" : "iso" }, - "data" : { "format" : "json" }, - "dimensionExclusions" : ["wp"] - } - }, - "plumber" : { - "type" : "realtime", - "windowPeriod" : "PT5m", - "segmentGranularity":"hour", - "basePersistDirectory" : "/tmp/realtime/basePersist", - "rejectionPolicy": { "type": "test" } - } -}] diff --git a/examples/bin/examples/rand/query.body b/examples/bin/examples/rand/query.body deleted file mode 100644 index 05007c955b9..00000000000 --- a/examples/bin/examples/rand/query.body +++ /dev/null @@ -1,19 +0,0 @@ -{ - "queryType": "groupBy", - "dataSource": "randSeq", - "granularity": "all", - "dimensions": [], - "aggregations":[ - { "type": "count", "name": "rows"}, - { "type": "doubleSum", "fieldName": "events", "name": "e"}, - { "type": "doubleSum", "fieldName": "outColumn", "name": "randomNumberSum"} - ], - "postAggregations":[ - { "type":"arithmetic", - "name":"avg_random", - "fn":"/", - "fields":[ {"type":"fieldAccess","name":"randomNumberSum","fieldName":"randomNumberSum"}, - {"type":"fieldAccess","name":"rows","fieldName":"rows"} ]} - ], - "intervals":["2012-10-01T00:00/2020-01-01T00"] -} diff --git a/examples/bin/examples/rand/rand_realtime.spec b/examples/bin/examples/rand/rand_realtime.spec deleted file mode 100644 index e30afbde877..00000000000 --- a/examples/bin/examples/rand/rand_realtime.spec +++ /dev/null @@ -1,32 +0,0 @@ -[{ - "schema": { - "dataSource": "randseq", - "aggregators": [ - {"type": "count", "name": "events"}, - {"type": "doubleSum", "name": "outColumn", "fieldName": "inColumn"} - ], - "indexGranularity": "minute", - "shardSpec": {"type": "none"} - }, - - "config": { - "maxRowsInMemory": 50000, - "intermediatePersistPeriod": "PT10m" - }, - - "firehose": { - "type": "rand", - "sleepUsec": 100000, - "maxGeneratedRows": 5000000, - "seed": 0, - "nTokens": 255, - "nPerSleep": 3 - }, - - "plumber": { - "type": "realtime", - "windowPeriod": "PT5m", - "segmentGranularity": "hour", - "basePersistDirectory": "/tmp/example/rand_realtime/basePersist" - } -}] diff --git a/examples/bin/examples/twitter/query.body b/examples/bin/examples/twitter/query.body index e0607aa1554..b8d4736b190 100644 --- a/examples/bin/examples/twitter/query.body +++ b/examples/bin/examples/twitter/query.body @@ -1,12 +1,24 @@ { - "queryType": "groupBy", - "dataSource": "twitterstream", - "granularity": "all", - "dimensions": ["lang", "utc_offset"], - "aggregations":[ - { "type": "count", "name": "rows"}, - { "type": "doubleSum", "fieldName": "tweets", "name": "tweets"} - ], - "filter": { "type": "selector", "dimension": "lang", "value": "en" }, - "intervals":["2012-10-01T00:00/2020-01-01T00"] -} + "queryType": "timeseries", + "dataSource": "twitterstream", + "granularity": "all", + "aggregations": [ + { + "type": "count", + "name": "rows" + }, + { + "type": "doubleSum", + "fieldName": "tweets", + "name": "tweets" + } + ], + "filter": { + "type": "selector", + "dimension": "lang", + "value": "en" + }, + "intervals": [ + "2012-10-01T00:00\/2020-01-01T00" + ] +} \ No newline at end of file diff --git a/examples/bin/examples/twitter/search_query.body b/examples/bin/examples/twitter/search_query.body index f2feb27d4fe..32716deaed4 100644 --- a/examples/bin/examples/twitter/search_query.body +++ b/examples/bin/examples/twitter/search_query.body @@ -1,13 +1,21 @@ { - "queryType": "search", - "dataSource": "twitterstream", - "granularity": "all", - "searchDimensions": ["htags"], - "limit": 1, - "query": { - "type": "fragment", - "values": ["men"], - "sort": { "type": "strlen" } - }, - "intervals":["2012-10-01T00:00/2020-01-01T00"] -} + "queryType": "search", + "dataSource": "twitterstream", + "granularity": "all", + "searchDimensions": [ + "htags" + ], + "limit": 1, + "query": { + "type": "fragment", + "values": [ + "men" + ], + "sort": { + "type": "strlen" + } + }, + "intervals": [ + "2012-10-01T00:00\/2020-01-01T00" + ] +} \ No newline at end of file diff --git a/examples/bin/examples/twitter/twitter_realtime.spec b/examples/bin/examples/twitter/twitter_realtime.spec index f17aab07f17..5b56351662c 100644 --- a/examples/bin/examples/twitter/twitter_realtime.spec +++ b/examples/bin/examples/twitter/twitter_realtime.spec @@ -1,44 +1,119 @@ -[{ - "schema": { - "dataSource": "twitterstream", - "aggregators": [ - {"type": "count", "name": "tweets"}, - {"type": "doubleSum", "fieldName": "follower_count", "name": "total_follower_count"}, - {"type": "doubleSum", "fieldName": "retweet_count", "name": "total_retweet_count" }, - {"type": "doubleSum", "fieldName": "friends_count", "name": "total_friends_count" }, - {"type": "doubleSum", "fieldName": "statuses_count", "name": "total_statuses_count"}, - - {"type": "min", "fieldName": "follower_count", "name": "min_follower_count"}, - {"type": "max", "fieldName": "follower_count", "name": "max_follower_count"}, - - {"type": "min", "fieldName": "friends_count", "name": "min_friends_count"}, - {"type": "max", "fieldName": "friends_count", "name": "max_friends_count"}, - - {"type": "min", "fieldName": "statuses_count", "name": "min_statuses_count"}, - {"type": "max", "fieldName": "statuses_count", "name": "max_statuses_count"}, - - {"type": "min", "fieldName": "retweet_count", "name": "min_retweet_count"}, - {"type": "max", "fieldName": "retweet_count", "name": "max_retweet_count"} - ], - "indexGranularity": "minute", - "shardSpec": {"type": "none"} +[ + { + "dataSchema": { + "dataSource": "twitterstream", + "parser": { + "parseSpec": { + "format": "json", + "timestampSpec": { + "column": "utcdt", + "format": "iso" + }, + "dimensionsSpec": { + "dimensions": [ + + ], + "dimensionExclusions": [ + + ], + "spatialDimensions": [ + + ] + } + } + }, + "metricsSpec": [ + { + "type": "count", + "name": "tweets" + }, + { + "type": "doubleSum", + "fieldName": "follower_count", + "name": "total_follower_count" + }, + { + "type": "doubleSum", + "fieldName": "retweet_count", + "name": "total_retweet_count" + }, + { + "type": "doubleSum", + "fieldName": "friends_count", + "name": "total_friends_count" + }, + { + "type": "doubleSum", + "fieldName": "statuses_count", + "name": "total_statuses_count" + }, + { + "type": "min", + "fieldName": "follower_count", + "name": "min_follower_count" + }, + { + "type": "max", + "fieldName": "follower_count", + "name": "max_follower_count" + }, + { + "type": "min", + "fieldName": "friends_count", + "name": "min_friends_count" + }, + { + "type": "max", + "fieldName": "friends_count", + "name": "max_friends_count" + }, + { + "type": "min", + "fieldName": "statuses_count", + "name": "min_statuses_count" + }, + { + "type": "max", + "fieldName": "statuses_count", + "name": "max_statuses_count" + }, + { + "type": "min", + "fieldName": "retweet_count", + "name": "min_retweet_count" + }, + { + "type": "max", + "fieldName": "retweet_count", + "name": "max_retweet_count" + } + ], + "granularitySpec": { + "type": "uniform", + "segmentGranularity": "DAY", + "queryGranularity": "NONE" + } }, - - "config": { - "maxRowsInMemory": 50000, - "intermediatePersistPeriod": "PT2m" - }, - - "firehose": { + "ioConfig": { + "type": "realtime", + "firehose": { "type": "twitzer", "maxEventCount": 500000, "maxRunMinutes": 120 + }, + "plumber": { + "type": "realtime" + } }, - - "plumber": { - "type": "realtime", - "windowPeriod": "PT3m", - "segmentGranularity": "hour", - "basePersistDirectory": "/tmp/example/twitter_realtime/basePersist" + "tuningConfig": { + "type": "realtime", + "maxRowsInMemory": 500000, + "intermediatePersistPeriod": "PT2m", + "windowPeriod": "PT3m", + "basePersistDirectory": "\/tmp\/realtime\/basePersist", + "rejectionPolicy": { + "type": "messageTime" + } } -}] + } +] \ No newline at end of file diff --git a/examples/bin/examples/webstream/query.body b/examples/bin/examples/webstream/query.body deleted file mode 100644 index 68189c27733..00000000000 --- a/examples/bin/examples/webstream/query.body +++ /dev/null @@ -1,27 +0,0 @@ -{ - "queryType":"groupBy", - "dataSource":"webstream", - "granularity":"minute", - "dimensions":[ - "timezone" - ], - "aggregations":[ - { - "type":"count", - "name":"rows" - }, - { - "type":"doubleSum", - "fieldName":"known_users", - "name":"known_users" - } - ], - "filter":{ - "type":"selector", - "dimension":"country", - "value":"US" - }, - "intervals":[ - "2013-06-01T00:00/2020-01-01T00" - ] -} diff --git a/examples/bin/examples/webstream/webstream_realtime.spec b/examples/bin/examples/webstream/webstream_realtime.spec deleted file mode 100644 index 2e2939a043c..00000000000 --- a/examples/bin/examples/webstream/webstream_realtime.spec +++ /dev/null @@ -1,47 +0,0 @@ -[{ - "schema": { - "dataSource": "webstream", - "aggregators": [ - {"type": "count", "name": "rows"}, - {"type": "doubleSum", "fieldName": "known_users", "name": "known_users"} - ], - "indexGranularity": "second", - "shardSpec": {"type": "none"} - }, - - "config": { - "maxRowsInMemory": 50000, - "intermediatePersistPeriod": "PT2m" - }, - - "firehose": { - "type": "webstream", - "url":"http://developer.usa.gov/1usagov", - "renamedDimensions": { - "g":"bitly_hash", - "c":"country", - "a":"user", - "cy":"city", - "l":"encoding_user_login", - "hh":"short_url", - "hc":"timestamp_hash", - "h":"user_bitly_hash", - "u":"url", - "tz":"timezone", - "t":"time", - "r":"referring_url", - "gr":"geo_region", - "nk":"known_users", - "al":"accept_language" - }, - "timeDimension":"t", - "timeFormat":"posix" - }, - - "plumber": { - "type": "realtime", - "windowPeriod": "PT3m", - "segmentGranularity": "hour", - "basePersistDirectory": "/tmp/example/usagov_realtime/basePersist" - } -}] diff --git a/examples/bin/examples/wikipedia/query.body b/examples/bin/examples/wikipedia/query.body index 32a65674868..8ff00386a05 100644 --- a/examples/bin/examples/wikipedia/query.body +++ b/examples/bin/examples/wikipedia/query.body @@ -1,10 +1,7 @@ { - "queryType":"groupBy", + "queryType":"timeseries", "dataSource":"wikipedia", "granularity":"minute", - "dimensions":[ - "page" - ], "aggregations":[ { "type":"count", diff --git a/examples/bin/examples/wikipedia/wikipedia_realtime.spec b/examples/bin/examples/wikipedia/wikipedia_realtime.spec index 3ed884ac9ea..93e4409ad38 100644 --- a/examples/bin/examples/wikipedia/wikipedia_realtime.spec +++ b/examples/bin/examples/wikipedia/wikipedia_realtime.spec @@ -1,63 +1,110 @@ [{ - "schema": { - "dataSource": "wikipedia", - "aggregators": [ - {"type": "count", "name": "count"}, - {"type": "longSum", "fieldName": "added", "name": "added"}, - {"type": "longSum", "fieldName": "deleted", "name": "deleted"}, - {"type": "longSum", "fieldName": "delta", "name": "delta"} - ], - "indexGranularity": "minute", - "shardSpec": {"type": "none"} - }, - - "config": { - "maxRowsInMemory": 50000, - "intermediatePersistPeriod": "PT2m" - }, - - "firehose": { - "type": "irc", - "nick": "wiki1234567890", - "host": "irc.wikimedia.org", - "channels": [ - "#en.wikipedia", - "#fr.wikipedia", - "#de.wikipedia", - "#ja.wikipedia" - ], - "decoder": { - "type": "wikipedia", - "namespaces": { - "#en.wikipedia": { - "": "main", - "Category": "category", - "$1 talk": "project talk", - "Template talk": "template talk", - "Help talk": "help talk", - "Media": "media", - "MediaWiki talk": "mediawiki talk", - "File talk": "file talk", - "MediaWiki": "mediawiki", - "User": "user", - "File": "file", - "User talk": "user talk", - "Template": "template", - "Help": "help", - "Special": "special", - "Talk": "talk", - "Category talk": "category talk" - } - } + "dataSchema": { + "dataSource": "wikipedia", + "parser": { + "type": "irc", + "parseSpec": { + "format": "json", + "timestampSpec": { + "column": "timestamp", + "format": "iso" }, - "timeDimension":"timestamp", - "timeFormat":"iso" + "dimensionsSpec": { + "dimensions": [ + "page", + "language", + "user", + "unpatrolled", + "newPage", + "robot", + "anonymous", + "namespace", + "continent", + "country", + "region", + "city" + ], + "dimensionExclusions": [], + "spatialDimensions": [] + } + }, + "decoder": { + "type": "wikipedia", + "namespaces": { + "#en.wikipedia": { + "_empty_": "main", + "Category": "category", + "$1 talk": "project talk", + "Template talk": "template talk", + "Help talk": "help talk", + "Media": "media", + "MediaWiki talk": "mediawiki talk", + "File talk": "file talk", + "MediaWiki": "mediawiki", + "User": "user", + "File": "file", + "User talk": "user talk", + "Template": "template", + "Help": "help", + "Special": "special", + "Talk": "talk", + "Category talk": "category talk" + } + } + } }, - - "plumber": { - "type": "realtime", - "windowPeriod": "PT3m", - "segmentGranularity": "hour", - "basePersistDirectory": "/tmp/example/wikipedia/basePersist" + "metricsSpec": [ + { + "type": "count", + "name": "count" + }, + { + "type": "doubleSum", + "name": "added", + "fieldName": "added" + }, + { + "type": "doubleSum", + "name": "deleted", + "fieldName": "deleted" + }, + { + "type": "doubleSum", + "name": "delta", + "fieldName": "delta" + } + ], + "granularitySpec": { + "type": "uniform", + "segmentGranularity": "DAY", + "queryGranularity": "NONE" } + }, + "ioConfig": { + "type": "realtime", + "firehose": { + "type": "irc", + "nick": "wiki1234567890", + "host": "irc.wikimedia.org", + "channels": [ + "#en.wikipedia", + "#fr.wikipedia", + "#de.wikipedia", + "#ja.wikipedia" + ] + }, + "plumber": { + "type": "realtime" + } + }, + "tuningConfig": { + "type": "realtime", + "maxRowsInMemory": 500000, + "intermediatePersistPeriod": "PT10m", + "windowPeriod": "PT10m", + "basePersistDirectory": "\/tmp\/realtime\/basePersist", + "rejectionPolicy": { + "type": "serverTime" + } + } }] diff --git a/examples/bin/run_example_client.sh b/examples/bin/run_example_client.sh index 5212adf2a52..a3c484d6efe 100755 --- a/examples/bin/run_example_client.sh +++ b/examples/bin/run_example_client.sh @@ -45,7 +45,7 @@ for delay in 5 30 30 30 30 30 30 30 30 30 30 echo "sleep for $delay seconds..." echo " " sleep $delay - curl -X POST 'http://localhost:8083/druid/v2/?w' -H 'content-type: application/json' -d "`cat ${QUERY_FILE}`" + curl -X POST 'http://localhost:8083/druid/v2/?pretty' -H 'content-type: application/json' -d "`cat ${QUERY_FILE}`" echo " " echo " " done diff --git a/examples/bin/run_example_server.sh b/examples/bin/run_example_server.sh index 461f937beb0..76aa2fae07f 100755 --- a/examples/bin/run_example_server.sh +++ b/examples/bin/run_example_server.sh @@ -58,7 +58,7 @@ DRUID_CP=${EXAMPLE_LOC} DRUID_CP=${DRUID_CP}:${SCRIPT_DIR}/../config/realtime #For the kit DRUID_CP=${DRUID_CP}:${SCRIPT_DIR}/lib/* -DRUID_CP=${DRUID_CP}:${SCRIPT_DIR}/config/_global +DRUID_CP=${DRUID_CP}:${SCRIPT_DIR}/config/_common DRUID_CP=${DRUID_CP}:${SCRIPT_DIR}/config/realtime echo "Running command:" diff --git a/examples/cassandra/README.md b/examples/cassandra/README.md deleted file mode 100644 index d29e9d8591a..00000000000 --- a/examples/cassandra/README.md +++ /dev/null @@ -1,5 +0,0 @@ -## Example Prerequisite - -The code in this example assumes Cassandra has been configured as deep storage for Druid. - -For details on how to accomplish this, see [Cassandra Deep Storage](http://druid.io/docs/latest/Cassandra-Deep-Storage.html). diff --git a/examples/cassandra/client.sh b/examples/cassandra/client.sh deleted file mode 100755 index 5ecb6d6cede..00000000000 --- a/examples/cassandra/client.sh +++ /dev/null @@ -1 +0,0 @@ -curl -sX POST "http://localhost:9090/druid/v2/?pretty=true" -H 'content-type: application/json' -d @query diff --git a/examples/cassandra/query b/examples/cassandra/query deleted file mode 100644 index 09aa4d3ce5d..00000000000 --- a/examples/cassandra/query +++ /dev/null @@ -1,19 +0,0 @@ -{ - "queryType": "groupBy", - "dataSource": "randSeq", - "granularity": "all", - "dimensions": [], - "aggregations":[ - { "type": "count", "name": "rows"}, - { "type": "doubleSum", "fieldName": "events", "name": "e"}, - { "type": "doubleSum", "fieldName": "outColumn", "name": "randomNumberSum"} - ], - "postAggregations":[ - { "type":"arithmetic", - "name":"avg_random", - "fn":"/", - "fields":[ {"type":"fieldAccess","name":"randomNumberSum","fieldName":"randomNumberSum"}, - {"type":"fieldAccess","name":"rows","fieldName":"rows"} ]} - ], - "intervals":["2012-10-01T00:00/2020-01-01T00"] - } diff --git a/examples/cassandra/schema/druid_schema.cql b/examples/cassandra/schema/druid_schema.cql deleted file mode 100644 index a38106d9715..00000000000 --- a/examples/cassandra/schema/druid_schema.cql +++ /dev/null @@ -1,2 +0,0 @@ -CREATE TABLE index_storage ( key text, chunk text, value blob, PRIMARY KEY (key, chunk)) WITH COMPACT STORAGE; -CREATE TABLE descriptor_storage ( key varchar, lastModified timestamp, descriptor varchar, PRIMARY KEY (key) ) WITH COMPACT STORAGE; diff --git a/examples/config/_global/global.runtime.properties b/examples/config/_common/common.runtime.properties similarity index 83% rename from examples/config/_global/global.runtime.properties rename to examples/config/_common/common.runtime.properties index 34a91eacc5c..c73d92bec8c 100644 --- a/examples/config/_global/global.runtime.properties +++ b/examples/config/_common/common.runtime.properties @@ -1,5 +1,5 @@ # Extensions -druid.extensions.coordinates=["io.druid.extensions:druid-examples:0.7.0","io.druid.extensions:druid-kafka-seven:0.7.0","io.druid.extensions:druid-rabbitmq:0.7.0", "io.druid.extensions:druid-s3-extensions:0.7.0"] +druid.extensions.coordinates=["io.druid.extensions:druid-examples:0.7.0-SNAPSHOT","io.druid.extensions:druid-kafka-seven:0.7.0-SNAPSHOT","io.druid.extensions:druid-rabbitmq:0.7.0-SNAPSHOT"] # Zookeeper druid.zk.service.host=localhost diff --git a/examples/config/historical/runtime.properties b/examples/config/historical/runtime.properties index 544e5c860e1..332aa0c95b0 100644 --- a/examples/config/historical/runtime.properties +++ b/examples/config/historical/runtime.properties @@ -2,16 +2,9 @@ druid.host=localhost druid.service=historical druid.port=8081 -druid.extensions.coordinates=["io.druid.extensions:druid-s3-extensions:0.7.0"] - -# Dummy read only AWS account (used to download example data) -druid.s3.secretKey=QyyfVZ7llSiRg6Qcrql1eEUG7buFpAK6T6engr1b -druid.s3.accessKey=AKIAIMKECRUYKDQGR6YQ - -druid.server.maxSize=10000000000 - # Change these to make Druid faster druid.processing.buffer.sizeBytes=100000000 druid.processing.numThreads=1 druid.segmentCache.locations=[{"path": "/tmp/druid/indexCache", "maxSize"\: 10000000000}] +druid.server.maxSize=10000000000 diff --git a/examples/src/main/java/io/druid/examples/ExamplesDruidModule.java b/examples/src/main/java/io/druid/examples/ExamplesDruidModule.java index 50ff21d3987..aa265a0462c 100644 --- a/examples/src/main/java/io/druid/examples/ExamplesDruidModule.java +++ b/examples/src/main/java/io/druid/examples/ExamplesDruidModule.java @@ -23,9 +23,7 @@ import com.fasterxml.jackson.databind.Module; import com.fasterxml.jackson.databind.jsontype.NamedType; import com.fasterxml.jackson.databind.module.SimpleModule; import com.google.inject.Binder; -import io.druid.examples.rand.RandomFirehoseFactory; import io.druid.examples.twitter.TwitterSpritzerFirehoseFactory; -import io.druid.examples.web.WebFirehoseFactory; import io.druid.initialization.DruidModule; import java.util.Arrays; @@ -41,9 +39,7 @@ public class ExamplesDruidModule implements DruidModule return Arrays.asList( new SimpleModule("ExamplesModule") .registerSubtypes( - new NamedType(TwitterSpritzerFirehoseFactory.class, "twitzer"), - new NamedType(RandomFirehoseFactory.class, "rand"), - new NamedType(WebFirehoseFactory.class, "webstream") + new NamedType(TwitterSpritzerFirehoseFactory.class, "twitzer") ) ); } diff --git a/examples/src/main/java/io/druid/examples/rand/RandomFirehoseFactory.java b/examples/src/main/java/io/druid/examples/rand/RandomFirehoseFactory.java deleted file mode 100644 index 91deffacc67..00000000000 --- a/examples/src/main/java/io/druid/examples/rand/RandomFirehoseFactory.java +++ /dev/null @@ -1,269 +0,0 @@ -/* - * 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.examples.rand; - -import com.fasterxml.jackson.annotation.JsonCreator; -import com.fasterxml.jackson.annotation.JsonProperty; -import com.fasterxml.jackson.annotation.JsonTypeName; -import com.google.common.collect.Maps; -import com.metamx.common.logger.Logger; -import io.druid.data.input.Firehose; -import io.druid.data.input.FirehoseFactory; -import io.druid.data.input.InputRow; -import io.druid.data.input.MapBasedInputRow; -import io.druid.data.input.impl.InputRowParser; - -import java.io.IOException; -import java.util.LinkedList; -import java.util.Map; -import java.util.Random; - -import static java.lang.Thread.sleep; - -/** - * Random value sequence Firehost Factory named "rand". - * Builds a Firehose that emits a stream of random numbers (outColumn, a positive double) - * with timestamps along with an associated token (target). This provides a timeseries - * that requires no network access for demonstration, characterization, and testing. - * The generated tuples can be thought of as asynchronously - * produced triples (timestamp, outColumn, target) where the timestamp varies depending on - * speed of processing. - * - *

- * InputRows are produced as fast as requested, so this can be used to determine the - * upper rate of ingest if sleepUsec is set to 0; nTokens specifies how many associated - * target labels are used. Generation is round-robin for nTokens and sleep occurs - * every nPerSleep values generated. A random number seed can be used by setting the - * firehose parameter "seed" to a non-zero value so that values can be reproducible - * (but note that timestamp is not deterministic because timestamps are obtained at - * the moment an event is delivered.) - * Values are offset by adding the modulus of the token number to the random number - * so that token values have distinct, non-overlapping ranges. - *

- * - * Example spec file: - *
- * [{
- * "schema" : { "dataSource":"randseq",
- * "aggregators":[ {"type":"count", "name":"events"},
- * {"type":"doubleSum","name":"outColumn","fieldName":"inColumn"} ],
- * "indexGranularity":"minute",
- * "shardSpec" : { "type": "none" } },
- * "config" : { "maxRowsInMemory" : 50000,
- * "intermediatePersistPeriod" : "PT2m" },
- *
- * "firehose" : { "type" : "rand",
- * "sleepUsec": 100000,
- * "maxGeneratedRows" : 5000000,
- * "seed" : 0,
- * "nTokens" : 19,
- * "nPerSleep" : 3
- * },
- *
- * "plumber" : { "type" : "realtime",
- * "windowPeriod" : "PT5m",
- * "segmentGranularity":"hour",
- * "basePersistDirectory" : "/tmp/realtime/basePersist" }
- * }]
- * 
- * - * Example query using POST to /druid/v2/ (where UTC date and time MUST include the current hour): - *
- * {
- * "queryType": "groupBy",
- * "dataSource": "randSeq",
- * "granularity": "all",
- * "dimensions": [],
- * "aggregations":[
- * { "type": "count", "name": "rows"},
- * { "type": "doubleSum", "fieldName": "events", "name": "e"},
- * { "type": "doubleSum", "fieldName": "outColumn", "name": "randomNumberSum"}
- * ],
- * "postAggregations":[
- * {  "type":"arithmetic",
- * "name":"avg_random",
- * "fn":"/",
- * "fields":[ {"type":"fieldAccess","name":"randomNumberSum","fieldName":"randomNumberSum"},
- * {"type":"fieldAccess","name":"rows","fieldName":"rows"} ]}
- * ],
- * "intervals":["2012-10-01T00:00/2020-01-01T00"]
- * }
- * 
- */ -@JsonTypeName("rand") -public class RandomFirehoseFactory implements FirehoseFactory -{ - private static final Logger log = new Logger(RandomFirehoseFactory.class); - /** - * msec to sleep before generating a new row; if this and delayNsec are 0, then go as fast as possible. - * json param sleepUsec (microseconds) is used to initialize this. - */ - private final long delayMsec; - /** - * nsec to sleep before generating a new row; if this and delayMsec are 0, then go as fast as possible. - * json param sleepUsec (microseconds) is used to initialize this. - */ - private final int delayNsec; - /** - * max rows to generate, -1 is infinite, 0 means nothing is generated; use this to prevent - * infinite space consumption or to see what happens when a Firehose stops delivering - * values, or to have hasMore() return false. - */ - private final long maxGeneratedRows; - /** - * seed for random number generator; if 0, then no seed is used. - */ - private final long seed; - /** - * number of tokens to randomly associate with values (no heap limits). This can be used to - * stress test the number of tokens. - */ - private final int nTokens; - /** - * Number of token events per sleep interval. - */ - private final int nPerSleep; - - @JsonCreator - public RandomFirehoseFactory( - @JsonProperty("sleepUsec") Long sleepUsec, - @JsonProperty("maxGeneratedRows") Long maxGeneratedRows, - @JsonProperty("seed") Long seed, - @JsonProperty("nTokens") Integer nTokens, - @JsonProperty("nPerSleep") Integer nPerSleep - ) - { - long nsec = (sleepUsec > 0) ? sleepUsec * 1000L : 0; - long msec = nsec / 1000000L; - this.delayMsec = msec; - this.delayNsec = (int) (nsec - (msec * 1000000L)); - this.maxGeneratedRows = maxGeneratedRows; - this.seed = seed; - this.nTokens = nTokens; - this.nPerSleep = nPerSleep; - if (nTokens <= 0) { - log.warn("nTokens parameter " + nTokens + " ignored; must be greater than or equal to 1"); - nTokens = 1; - } - if (nPerSleep <= 0) { - log.warn("nPerSleep parameter " + nPerSleep + " ignored; must be greater than or equal to 1"); - nPerSleep = 1; - } - log.info("maxGeneratedRows=" + maxGeneratedRows); - log.info("seed=" + ((seed == 0L) ? "random value" : seed)); - log.info("nTokens=" + nTokens); - log.info("nPerSleep=" + nPerSleep); - double dmsec = (double) delayMsec + ((double) this.delayNsec) / 1000000.; - if (dmsec > 0.0) { - log.info("sleep period=" + dmsec + "msec"); - log.info( - "approximate max rate of record generation=" + (nPerSleep * 1000. / dmsec) + "/sec" + - " or " + (60. * nPerSleep * 1000. / dmsec) + "/minute" - ); - } else { - log.info("sleep period= NONE"); - log.info("approximate max rate of record generation= as fast as possible"); - } - } - - @Override - public Firehose connect(InputRowParser parser) throws IOException - { - final LinkedList dimensions = new LinkedList(); - dimensions.add("inColumn"); - dimensions.add("target"); - - return new Firehose() - { - private final java.util.Random rand = (seed == 0L) ? new Random() : new Random(seed); - - private long rowCount = 0L; - private boolean waitIfmaxGeneratedRows = true; - - @Override - public boolean hasMore() - { - if (maxGeneratedRows >= 0 && rowCount >= maxGeneratedRows) { - return waitIfmaxGeneratedRows; - } else { - return true; // there are always more random numbers - } - } - - @Override - public InputRow nextRow() - { - final long modulus = rowCount % nPerSleep; - final long nth = (rowCount % nTokens) + 1; - long sleepMsec = delayMsec; - // all done? - if (maxGeneratedRows >= 0 && rowCount >= maxGeneratedRows && waitIfmaxGeneratedRows) { - // sleep a long time instead of terminating - sleepMsec = 2000000000L; - } - if (sleepMsec > 0L || delayNsec > 0) { - try { - if (modulus == 0) { - sleep(sleepMsec, delayNsec); - } - } - catch (InterruptedException e) { - throw new RuntimeException("InterruptedException"); - } - } - if (++rowCount % 1000 == 0) { - log.info("%,d events created.", rowCount); - } - - final Map theMap = Maps.newTreeMap(String.CASE_INSENSITIVE_ORDER); - theMap.put("inColumn", anotherRand((int) nth)); - theMap.put("target", ("a" + nth)); - return new MapBasedInputRow(System.currentTimeMillis(), dimensions, theMap); - } - - private Float anotherRand(int scale) - { - double f = rand.nextDouble(); // [0.0,1.0] - return new Float(f + (double) scale); - } - - @Override - public Runnable commit() - { - // Do nothing. - return new Runnable() - { - @Override - public void run() - { - - } - }; - } - - @Override - public void close() throws IOException - { - // do nothing - } - }; - } - -} diff --git a/examples/src/main/java/io/druid/examples/web/InputSupplierUpdateStream.java b/examples/src/main/java/io/druid/examples/web/InputSupplierUpdateStream.java deleted file mode 100644 index c558d8f998c..00000000000 --- a/examples/src/main/java/io/druid/examples/web/InputSupplierUpdateStream.java +++ /dev/null @@ -1,133 +0,0 @@ -/* - * 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.examples.web; - -import com.fasterxml.jackson.core.JsonParseException; -import com.fasterxml.jackson.core.type.TypeReference; -import com.fasterxml.jackson.databind.JsonMappingException; -import com.fasterxml.jackson.databind.ObjectMapper; -import com.google.common.io.InputSupplier; -import com.metamx.emitter.EmittingLogger; -import io.druid.jackson.DefaultObjectMapper; - -import java.io.BufferedReader; -import java.io.IOException; -import java.util.HashMap; -import java.util.Map; -import java.util.concurrent.ArrayBlockingQueue; -import java.util.concurrent.BlockingQueue; -import java.util.concurrent.TimeUnit; - -public class InputSupplierUpdateStream implements UpdateStream -{ - private static final EmittingLogger log = new EmittingLogger(InputSupplierUpdateStream.class); - private static final long queueWaitTime = 15L; - private final TypeReference> typeRef; - private final InputSupplier supplier; - private final int QUEUE_SIZE = 10000; - private final BlockingQueue> queue = new ArrayBlockingQueue>(QUEUE_SIZE); - private final ObjectMapper mapper = new DefaultObjectMapper(); - private final String timeDimension; - private final Thread addToQueueThread; - - public InputSupplierUpdateStream( - final InputSupplier supplier, - final String timeDimension - ) - { - addToQueueThread = new Thread() - { - public void run() - { - while (!isInterrupted()) { - try { - BufferedReader reader = supplier.getInput(); - String line; - while ((line = reader.readLine()) != null) { - if (isValid(line)) { - HashMap map = mapper.readValue(line, typeRef); - if (map.get(timeDimension) != null) { - queue.offer(map, queueWaitTime, TimeUnit.SECONDS); - log.debug("Successfully added to queue"); - } else { - log.info("missing timestamp"); - } - } - } - } - - catch (InterruptedException e){ - log.info(e, "Thread adding events to the queue interrupted"); - return; - } - catch (JsonMappingException e) { - log.info(e, "Error in converting json to map"); - } - catch (JsonParseException e) { - log.info(e, "Error in parsing json"); - } - catch (IOException e) { - log.info(e, "Error in connecting to InputStream"); - } - } - } - }; - addToQueueThread.setDaemon(true); - - this.supplier = supplier; - this.typeRef = new TypeReference>() - { - }; - this.timeDimension = timeDimension; - } - - private boolean isValid(String s) - { - return !(s.isEmpty()); - } - - public void start() - { - addToQueueThread.start(); - - } - - public void stop() - { - addToQueueThread.interrupt(); - } - - - public Map pollFromQueue(long waitTime, TimeUnit unit) throws InterruptedException - { - return queue.poll(waitTime, unit); - } - - public int getQueueSize() - { - return queue.size(); - } - - public String getTimeDimension() - { - return timeDimension; - } - -} diff --git a/examples/src/main/java/io/druid/examples/web/InputSupplierUpdateStreamFactory.java b/examples/src/main/java/io/druid/examples/web/InputSupplierUpdateStreamFactory.java deleted file mode 100644 index 6741ec7b870..00000000000 --- a/examples/src/main/java/io/druid/examples/web/InputSupplierUpdateStreamFactory.java +++ /dev/null @@ -1,42 +0,0 @@ -/* - * Druid - a distributed column store. - * Copyright (C) 2012 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.examples.web; - -import com.google.common.io.InputSupplier; - -import java.io.BufferedReader; - -public class InputSupplierUpdateStreamFactory implements UpdateStreamFactory -{ - private final InputSupplier inputSupplier; - private final String timeDimension; - - public InputSupplierUpdateStreamFactory(InputSupplier inputSupplier, String timeDimension) - { - this.inputSupplier = inputSupplier; - this.timeDimension = timeDimension; - } - - public InputSupplierUpdateStream build() - { - return new InputSupplierUpdateStream(inputSupplier, timeDimension); - } - -} diff --git a/examples/src/main/java/io/druid/examples/web/RenamingKeysUpdateStream.java b/examples/src/main/java/io/druid/examples/web/RenamingKeysUpdateStream.java deleted file mode 100644 index fd35aa3a233..00000000000 --- a/examples/src/main/java/io/druid/examples/web/RenamingKeysUpdateStream.java +++ /dev/null @@ -1,81 +0,0 @@ -/* - * Druid - a distributed column store. - * Copyright (C) 2012 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.examples.web; - -import com.google.common.collect.Maps; - -import java.util.Map; -import java.util.concurrent.TimeUnit; - -public class RenamingKeysUpdateStream implements UpdateStream -{ - - private final InputSupplierUpdateStream updateStream; - private Map renamedDimensions; - - public RenamingKeysUpdateStream( - InputSupplierUpdateStream updateStream, - Map renamedDimensions - ) - { - this.renamedDimensions = renamedDimensions; - this.updateStream = updateStream; - } - - public Map pollFromQueue(long waitTime, TimeUnit unit) throws InterruptedException - { - return renameKeys(updateStream.pollFromQueue(waitTime, unit)); - } - - - private Map renameKeys(Map update) - { - if (renamedDimensions != null) { - Map renamedMap = Maps.newHashMap(); - for (String key : renamedDimensions.keySet()) { - if (update.get(key) != null) { - Object obj = update.get(key); - renamedMap.put(renamedDimensions.get(key), obj); - } - } - return renamedMap; - } else { - return update; - } - } - - public String getTimeDimension() - { - if (renamedDimensions != null && renamedDimensions.get(updateStream.getTimeDimension()) != null) { - return renamedDimensions.get(updateStream.getTimeDimension()); - } - return updateStream.getTimeDimension(); - - } - - public void start() - { - updateStream.start(); - } - - public void stop(){ - updateStream.stop(); - } -} diff --git a/examples/src/main/java/io/druid/examples/web/RenamingKeysUpdateStreamFactory.java b/examples/src/main/java/io/druid/examples/web/RenamingKeysUpdateStreamFactory.java deleted file mode 100644 index f30566041a8..00000000000 --- a/examples/src/main/java/io/druid/examples/web/RenamingKeysUpdateStreamFactory.java +++ /dev/null @@ -1,39 +0,0 @@ -/* - * Druid - a distributed column store. - * Copyright (C) 2012 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.examples.web; - -import java.util.Map; - -public class RenamingKeysUpdateStreamFactory implements UpdateStreamFactory -{ - private InputSupplierUpdateStreamFactory updateStreamFactory; - private Map renamedDimensions; - - public RenamingKeysUpdateStreamFactory(InputSupplierUpdateStreamFactory updateStreamFactory, Map renamedDimensions) - { - this.updateStreamFactory = updateStreamFactory; - this.renamedDimensions = renamedDimensions; - } - - public RenamingKeysUpdateStream build() - { - return new RenamingKeysUpdateStream(updateStreamFactory.build(), renamedDimensions); - } -} diff --git a/examples/src/main/java/io/druid/examples/web/UpdateStream.java b/examples/src/main/java/io/druid/examples/web/UpdateStream.java deleted file mode 100644 index 0d64e8bde36..00000000000 --- a/examples/src/main/java/io/druid/examples/web/UpdateStream.java +++ /dev/null @@ -1,31 +0,0 @@ -/* - * Druid - a distributed column store. - * Copyright (C) 2012 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.examples.web; - -import java.util.Map; -import java.util.concurrent.TimeUnit; - -public interface UpdateStream -{ - public Map pollFromQueue(long waitTime, TimeUnit unit) throws InterruptedException; - public String getTimeDimension(); - public void start(); - public void stop(); - -} diff --git a/examples/src/main/java/io/druid/examples/web/UpdateStreamFactory.java b/examples/src/main/java/io/druid/examples/web/UpdateStreamFactory.java deleted file mode 100644 index 8c7831237d7..00000000000 --- a/examples/src/main/java/io/druid/examples/web/UpdateStreamFactory.java +++ /dev/null @@ -1,24 +0,0 @@ -/* - * Druid - a distributed column store. - * Copyright (C) 2012 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.examples.web; - -public interface UpdateStreamFactory -{ - public UpdateStream build(); -} diff --git a/examples/src/main/java/io/druid/examples/web/WebFirehoseFactory.java b/examples/src/main/java/io/druid/examples/web/WebFirehoseFactory.java deleted file mode 100644 index 76301512e20..00000000000 --- a/examples/src/main/java/io/druid/examples/web/WebFirehoseFactory.java +++ /dev/null @@ -1,136 +0,0 @@ -/* - * 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.examples.web; - -import com.fasterxml.jackson.annotation.JsonCreator; -import com.fasterxml.jackson.annotation.JsonProperty; -import com.fasterxml.jackson.annotation.JsonTypeName; -import com.google.common.base.Throwables; -import com.metamx.common.parsers.TimestampParser; -import com.metamx.emitter.EmittingLogger; -import io.druid.data.input.Firehose; -import io.druid.data.input.FirehoseFactory; -import io.druid.data.input.InputRow; -import io.druid.data.input.MapBasedInputRow; -import io.druid.data.input.impl.InputRowParser; -import io.druid.utils.Runnables; -import org.joda.time.DateTime; - -import java.io.IOException; -import java.util.ArrayList; -import java.util.Map; -import java.util.concurrent.TimeUnit; - -@JsonTypeName("webstream") -public class WebFirehoseFactory implements FirehoseFactory -{ - private static final EmittingLogger log = new EmittingLogger(WebFirehoseFactory.class); - - private final String timeFormat; - private final UpdateStreamFactory factory; - private final long queueWaitTime = 15L; - - @JsonCreator - public WebFirehoseFactory( - @JsonProperty("url") String url, - @JsonProperty("renamedDimensions") Map renamedDimensions, - @JsonProperty("timeDimension") String timeDimension, - @JsonProperty("timeFormat") String timeFormat - ) - { - this( - new RenamingKeysUpdateStreamFactory( - new InputSupplierUpdateStreamFactory(new WebJsonSupplier(url), timeDimension), - renamedDimensions - ), timeFormat - ); - } - - public WebFirehoseFactory(UpdateStreamFactory factory, String timeFormat) - { - this.factory = factory; - if (timeFormat == null) { - this.timeFormat = "auto"; - } else { - this.timeFormat = timeFormat; - } - } - - @Override - public Firehose connect(InputRowParser parser) throws IOException - { - - final UpdateStream updateStream = factory.build(); - updateStream.start(); - - return new Firehose() - { - Map map; - private final Runnable doNothingRunnable = Runnables.getNoopRunnable(); - - @Override - public boolean hasMore() - { - try { - map = updateStream.pollFromQueue(queueWaitTime, TimeUnit.SECONDS); - return map != null; - } - catch (InterruptedException e) { - throw Throwables.propagate(e); - } - } - - - @Override - public InputRow nextRow() - { - try { - DateTime date = TimestampParser.createTimestampParser(timeFormat) - .apply(map.get(updateStream.getTimeDimension()).toString()); - return new MapBasedInputRow( - date.getMillis(), - new ArrayList(map.keySet()), - map - ); - } - catch (Exception e) { - throw Throwables.propagate(e); - } - finally { - map = null; - } - } - - @Override - public Runnable commit() - { - // ephemera in, ephemera out. - return doNothingRunnable; // reuse the same object each time - } - - @Override - public void close() throws IOException - { - updateStream.stop(); - } - - }; - } -} diff --git a/examples/src/main/java/io/druid/examples/web/WebJsonSupplier.java b/examples/src/main/java/io/druid/examples/web/WebJsonSupplier.java deleted file mode 100644 index 39d0fae1882..00000000000 --- a/examples/src/main/java/io/druid/examples/web/WebJsonSupplier.java +++ /dev/null @@ -1,61 +0,0 @@ -/* - * Druid - a distributed column store. - * Copyright (C) 2012 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.examples.web; - -import com.google.api.client.repackaged.com.google.common.base.Throwables; -import com.google.common.base.Charsets; -import com.google.common.base.Preconditions; -import com.google.common.io.InputSupplier; -import com.metamx.emitter.EmittingLogger; -import org.apache.commons.validator.routines.UrlValidator; - -import java.io.BufferedReader; -import java.io.IOException; -import java.io.InputStreamReader; -import java.net.URL; -import java.net.URLConnection; - -public class WebJsonSupplier implements InputSupplier -{ - private static final EmittingLogger log = new EmittingLogger(WebJsonSupplier.class); - private static final UrlValidator urlValidator = new UrlValidator(); - - private URL url; - - public WebJsonSupplier(String urlString) - { - Preconditions.checkState(urlValidator.isValid(urlString)); - - try { - this.url = new URL(urlString); - } - catch (Exception e) { - throw Throwables.propagate(e); - } - } - - @Override - public BufferedReader getInput() throws IOException - { - URLConnection connection = url.openConnection(); - connection.setDoInput(true); - return new BufferedReader(new InputStreamReader(url.openStream(), Charsets.UTF_8)); - } -} diff --git a/examples/src/test/java/io/druid/examples/web/InputSupplierUpdateStreamTest.java b/examples/src/test/java/io/druid/examples/web/InputSupplierUpdateStreamTest.java deleted file mode 100644 index d076afd4e9a..00000000000 --- a/examples/src/test/java/io/druid/examples/web/InputSupplierUpdateStreamTest.java +++ /dev/null @@ -1,114 +0,0 @@ -/* - * Druid - a distributed column store. - * Copyright (C) 2012 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.examples.web; - -import com.google.common.io.InputSupplier; -import junit.framework.Assert; -import org.junit.Before; -import org.junit.Test; - -import java.util.ArrayList; -import java.util.HashMap; -import java.util.Map; -import java.util.concurrent.TimeUnit; - -public class InputSupplierUpdateStreamTest -{ - private final long waitTime = 1L; - private final TimeUnit unit = TimeUnit.SECONDS; - private final ArrayList dimensions = new ArrayList(); - private InputSupplier testCaseSupplier; - Map expectedAnswer = new HashMap(); - String timeDimension; - - @Before - public void setUp() - { - timeDimension = "time"; - testCaseSupplier = new TestCaseSupplier( - "{\"item1\": \"value1\"," - + "\"item2\":2," - + "\"time\":1372121562 }" - ); - - dimensions.add("item1"); - dimensions.add("item2"); - dimensions.add("time"); - - expectedAnswer.put("item1", "value1"); - expectedAnswer.put("item2", 2); - expectedAnswer.put("time", 1372121562); - } - - - @Test - public void basicIngestionCheck() throws Exception - { - InputSupplierUpdateStream updateStream = new InputSupplierUpdateStream( - testCaseSupplier, - timeDimension - ); - updateStream.start(); - Map insertedRow = updateStream.pollFromQueue(waitTime, unit); - Assert.assertEquals(expectedAnswer, insertedRow); - updateStream.stop(); - } - - //If a timestamp is missing, we should throw away the event - @Test - public void missingTimeStampCheck() - { - testCaseSupplier = new TestCaseSupplier( - "{\"item1\": \"value1\"," - + "\"item2\":2}" - ); - - InputSupplierUpdateStream updateStream = new InputSupplierUpdateStream( - testCaseSupplier, - timeDimension - ); - updateStream.start(); - Assert.assertEquals(updateStream.getQueueSize(), 0); - updateStream.stop(); - } - - //If any other value is missing, we should still add the event and process it properly - @Test - public void otherNullValueCheck() throws Exception - { - testCaseSupplier = new TestCaseSupplier( - "{\"item1\": \"value1\"," - + "\"time\":1372121562 }" - ); - InputSupplierUpdateStream updateStream = new InputSupplierUpdateStream( - testCaseSupplier, - timeDimension - ); - updateStream.start(); - Map insertedRow = updateStream.pollFromQueue(waitTime, unit); - Map expectedAnswer = new HashMap(); - expectedAnswer.put("item1", "value1"); - expectedAnswer.put("time", 1372121562); - Assert.assertEquals(expectedAnswer, insertedRow); - updateStream.stop(); - } - - -} diff --git a/examples/src/test/java/io/druid/examples/web/RenamingKeysUpdateStreamTest.java b/examples/src/test/java/io/druid/examples/web/RenamingKeysUpdateStreamTest.java deleted file mode 100644 index dc08e7fe471..00000000000 --- a/examples/src/test/java/io/druid/examples/web/RenamingKeysUpdateStreamTest.java +++ /dev/null @@ -1,91 +0,0 @@ -/* - * Druid - a distributed column store. - * Copyright (C) 2012 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.examples.web; - -import com.google.common.io.InputSupplier; -import junit.framework.Assert; -import org.junit.Before; -import org.junit.Test; - -import java.util.HashMap; -import java.util.Map; -import java.util.concurrent.TimeUnit; - -public class RenamingKeysUpdateStreamTest -{ - private final long waitTime = 15L; - private final TimeUnit unit = TimeUnit.SECONDS; - private InputSupplier testCaseSupplier; - String timeDimension; - - @Before - public void setUp() - { - timeDimension = "time"; - testCaseSupplier = new TestCaseSupplier( - "{\"item1\": \"value1\"," - + "\"item2\":2," - + "\"time\":1372121562 }" - ); - } - - @Test - public void testPolFromQueue() throws Exception - { - InputSupplierUpdateStream updateStream = new InputSupplierUpdateStream(testCaseSupplier, timeDimension); - Map renamedKeys = new HashMap(); - renamedKeys.put("item1", "i1"); - renamedKeys.put("item2", "i2"); - renamedKeys.put("time", "t"); - - RenamingKeysUpdateStream renamer = new RenamingKeysUpdateStream(updateStream, renamedKeys); - renamer.start(); - Map expectedAnswer = new HashMap(); - expectedAnswer.put("i1", "value1"); - expectedAnswer.put("i2", 2); - expectedAnswer.put("t", 1372121562); - Assert.assertEquals(expectedAnswer, renamer.pollFromQueue(waitTime, unit)); - } - - @Test - public void testGetTimeDimension() throws Exception - { - InputSupplierUpdateStream updateStream = new InputSupplierUpdateStream(testCaseSupplier, timeDimension); - Map renamedKeys = new HashMap(); - renamedKeys.put("item1", "i1"); - renamedKeys.put("item2", "i2"); - renamedKeys.put("time", "t"); - - RenamingKeysUpdateStream renamer = new RenamingKeysUpdateStream(updateStream, renamedKeys); - Assert.assertEquals("t", renamer.getTimeDimension()); - } - - @Test - public void testMissingTimeRename() throws Exception - { - InputSupplierUpdateStream updateStream = new InputSupplierUpdateStream(testCaseSupplier, timeDimension); - Map renamedKeys = new HashMap(); - renamedKeys.put("item1", "i1"); - renamedKeys.put("item2", "i2"); - RenamingKeysUpdateStream renamer = new RenamingKeysUpdateStream(updateStream, renamedKeys); - Assert.assertEquals("time", renamer.getTimeDimension()); - } - -} diff --git a/examples/src/test/java/io/druid/examples/web/TestCaseSupplier.java b/examples/src/test/java/io/druid/examples/web/TestCaseSupplier.java deleted file mode 100644 index baf44a613d8..00000000000 --- a/examples/src/test/java/io/druid/examples/web/TestCaseSupplier.java +++ /dev/null @@ -1,42 +0,0 @@ -/* - * Druid - a distributed column store. - * Copyright (C) 2012 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.examples.web; - -import com.google.common.io.InputSupplier; - -import java.io.BufferedReader; -import java.io.IOException; -import java.io.StringReader; - -public class TestCaseSupplier implements InputSupplier -{ - private final String testString; - - public TestCaseSupplier(String testString) - { - this.testString = testString; - } - - @Override - public BufferedReader getInput() throws IOException - { - return new BufferedReader(new StringReader(testString)); - } -} diff --git a/examples/src/test/java/io/druid/examples/web/WebFirehoseFactoryTest.java b/examples/src/test/java/io/druid/examples/web/WebFirehoseFactoryTest.java deleted file mode 100644 index 9310a5f36f4..00000000000 --- a/examples/src/test/java/io/druid/examples/web/WebFirehoseFactoryTest.java +++ /dev/null @@ -1,223 +0,0 @@ -/* - * 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.examples.web; - -import com.google.common.collect.ImmutableMap; -import com.google.common.collect.Lists; -import io.druid.data.input.Firehose; -import io.druid.data.input.InputRow; -import org.joda.time.DateTime; -import org.junit.Assert; -import org.junit.Before; -import org.junit.Test; - -import java.util.Collections; -import java.util.List; -import java.util.Map; -import java.util.concurrent.TimeUnit; - -public class WebFirehoseFactoryTest -{ - private List dimensions = Lists.newArrayList(); - private WebFirehoseFactory webbie; - private WebFirehoseFactory webbie1; - - @Before - public void setUp() throws Exception - { - dimensions.add("item1"); - dimensions.add("item2"); - dimensions.add("time"); - webbie = new WebFirehoseFactory( - new UpdateStreamFactory() - { - @Override - public UpdateStream build() - { - return new MyUpdateStream(ImmutableMap.of("item1", "value1", "item2", 2, "time", "1372121562")); - } - }, - "posix" - ); - - webbie1 = new WebFirehoseFactory( - new UpdateStreamFactory() - { - @Override - public UpdateStream build() - { - return new MyUpdateStream(ImmutableMap.of("item1", "value1", "item2", 2, "time", "1373241600000")); - } - }, - "auto" - ); - - } - - @Test - public void testDimensions() throws Exception - { - InputRow inputRow; - Firehose firehose = webbie.connect(null); - if (firehose.hasMore()) { - inputRow = firehose.nextRow(); - } else { - throw new RuntimeException("queue is empty"); - } - List actualAnswer = inputRow.getDimensions(); - Collections.sort(actualAnswer); - Assert.assertEquals(actualAnswer, dimensions); - } - - @Test - public void testPosixTimeStamp() throws Exception - { - InputRow inputRow; - Firehose firehose = webbie.connect(null); - if (firehose.hasMore()) { - inputRow = firehose.nextRow(); - } else { - throw new RuntimeException("queue is empty"); - } - long expectedTime = 1372121562L * 1000L; - Assert.assertEquals(expectedTime, inputRow.getTimestampFromEpoch()); - } - - @Test - public void testISOTimeStamp() throws Exception - { - WebFirehoseFactory webbie3 = new WebFirehoseFactory( - new UpdateStreamFactory() - { - @Override - public UpdateStream build() - { - return new MyUpdateStream(ImmutableMap.of("item1", "value1", "item2", 2, "time", "2013-07-08")); - } - }, - "auto" - ); - Firehose firehose1 = webbie3.connect(null); - if (firehose1.hasMore()) { - long milliSeconds = firehose1.nextRow().getTimestampFromEpoch(); - DateTime date = new DateTime("2013-07-08"); - Assert.assertEquals(date.getMillis(), milliSeconds); - } else { - Assert.assertFalse("hasMore returned false", true); - } - } - - @Test - public void testAutoIsoTimeStamp() throws Exception - { - WebFirehoseFactory webbie2 = new WebFirehoseFactory( - new UpdateStreamFactory() - { - @Override - public UpdateStream build() - { - return new MyUpdateStream(ImmutableMap.of("item1", "value1", "item2", 2, "time", "2013-07-08")); - } - }, - null - ); - Firehose firehose2 = webbie2.connect(null); - if (firehose2.hasMore()) { - long milliSeconds = firehose2.nextRow().getTimestampFromEpoch(); - DateTime date = new DateTime("2013-07-08"); - Assert.assertEquals(date.getMillis(), milliSeconds); - } else { - Assert.assertFalse("hasMore returned false", true); - } - } - - @Test - public void testAutoMilliSecondsTimeStamp() throws Exception - { - Firehose firehose3 = webbie1.connect(null); - if (firehose3.hasMore()) { - long milliSeconds = firehose3.nextRow().getTimestampFromEpoch(); - DateTime date = new DateTime("2013-07-08"); - Assert.assertEquals(date.getMillis(), milliSeconds); - } else { - Assert.assertFalse("hasMore returned false", true); - } - } - - @Test - public void testGetDimension() throws Exception - { - InputRow inputRow; - Firehose firehose = webbie1.connect(null); - if (firehose.hasMore()) { - inputRow = firehose.nextRow(); - } else { - throw new RuntimeException("queue is empty"); - } - - List column1 = Lists.newArrayList(); - column1.add("value1"); - Assert.assertEquals(column1, inputRow.getDimension("item1")); - } - - @Test - public void testGetFloatMetric() throws Exception - { - InputRow inputRow; - Firehose firehose = webbie1.connect(null); - if (firehose.hasMore()) { - inputRow = firehose.nextRow(); - } else { - throw new RuntimeException("queue is empty"); - } - - Assert.assertEquals((float) 2.0, inputRow.getFloatMetric("item2"), 0.0f); - } - - private static class MyUpdateStream implements UpdateStream - { - private static ImmutableMap map; - public MyUpdateStream(ImmutableMap map){ - this.map=map; - } - - @Override - public Map pollFromQueue(long waitTime, TimeUnit unit) throws InterruptedException - { - return map; - } - - @Override - public String getTimeDimension() - { - return "time"; - } - - @Override - public void start() - { - } - - @Override - public void stop() - { - } - } -} diff --git a/examples/src/test/java/io/druid/examples/web/WebJsonSupplierTest.java b/examples/src/test/java/io/druid/examples/web/WebJsonSupplierTest.java deleted file mode 100644 index b9e8582a1d1..00000000000 --- a/examples/src/test/java/io/druid/examples/web/WebJsonSupplierTest.java +++ /dev/null @@ -1,32 +0,0 @@ -/* - * Druid - a distributed column store. - * Copyright (C) 2012 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.examples.web; - -import org.junit.Test; - -public class WebJsonSupplierTest -{ - @Test(expected = IllegalStateException.class) - public void checkInvalidUrl() throws Exception - { - String invalidURL = "http://invalid.url."; - WebJsonSupplier supplier = new WebJsonSupplier(invalidURL); - } -} diff --git a/cassandra-storage/pom.xml b/extensions/cassandra-storage/pom.xml similarity index 100% rename from cassandra-storage/pom.xml rename to extensions/cassandra-storage/pom.xml diff --git a/cassandra-storage/src/main/java/io/druid/storage/cassandra/CassandraDataSegmentConfig.java b/extensions/cassandra-storage/src/main/java/io/druid/storage/cassandra/CassandraDataSegmentConfig.java similarity index 100% rename from cassandra-storage/src/main/java/io/druid/storage/cassandra/CassandraDataSegmentConfig.java rename to extensions/cassandra-storage/src/main/java/io/druid/storage/cassandra/CassandraDataSegmentConfig.java diff --git a/cassandra-storage/src/main/java/io/druid/storage/cassandra/CassandraDataSegmentPuller.java b/extensions/cassandra-storage/src/main/java/io/druid/storage/cassandra/CassandraDataSegmentPuller.java similarity index 100% rename from cassandra-storage/src/main/java/io/druid/storage/cassandra/CassandraDataSegmentPuller.java rename to extensions/cassandra-storage/src/main/java/io/druid/storage/cassandra/CassandraDataSegmentPuller.java diff --git a/cassandra-storage/src/main/java/io/druid/storage/cassandra/CassandraDataSegmentPusher.java b/extensions/cassandra-storage/src/main/java/io/druid/storage/cassandra/CassandraDataSegmentPusher.java similarity index 100% rename from cassandra-storage/src/main/java/io/druid/storage/cassandra/CassandraDataSegmentPusher.java rename to extensions/cassandra-storage/src/main/java/io/druid/storage/cassandra/CassandraDataSegmentPusher.java diff --git a/cassandra-storage/src/main/java/io/druid/storage/cassandra/CassandraDruidModule.java b/extensions/cassandra-storage/src/main/java/io/druid/storage/cassandra/CassandraDruidModule.java similarity index 100% rename from cassandra-storage/src/main/java/io/druid/storage/cassandra/CassandraDruidModule.java rename to extensions/cassandra-storage/src/main/java/io/druid/storage/cassandra/CassandraDruidModule.java diff --git a/cassandra-storage/src/main/java/io/druid/storage/cassandra/CassandraStorage.java b/extensions/cassandra-storage/src/main/java/io/druid/storage/cassandra/CassandraStorage.java similarity index 100% rename from cassandra-storage/src/main/java/io/druid/storage/cassandra/CassandraStorage.java rename to extensions/cassandra-storage/src/main/java/io/druid/storage/cassandra/CassandraStorage.java diff --git a/cassandra-storage/src/main/resources/META-INF/services/io.druid.initialization.DruidModule b/extensions/cassandra-storage/src/main/resources/META-INF/services/io.druid.initialization.DruidModule similarity index 100% rename from cassandra-storage/src/main/resources/META-INF/services/io.druid.initialization.DruidModule rename to extensions/cassandra-storage/src/main/resources/META-INF/services/io.druid.initialization.DruidModule diff --git a/hdfs-storage/pom.xml b/extensions/hdfs-storage/pom.xml similarity index 100% rename from hdfs-storage/pom.xml rename to extensions/hdfs-storage/pom.xml diff --git a/hdfs-storage/src/main/java/io/druid/storage/hdfs/HdfsDataSegmentKiller.java b/extensions/hdfs-storage/src/main/java/io/druid/storage/hdfs/HdfsDataSegmentKiller.java similarity index 100% rename from hdfs-storage/src/main/java/io/druid/storage/hdfs/HdfsDataSegmentKiller.java rename to extensions/hdfs-storage/src/main/java/io/druid/storage/hdfs/HdfsDataSegmentKiller.java diff --git a/hdfs-storage/src/main/java/io/druid/storage/hdfs/HdfsDataSegmentPuller.java b/extensions/hdfs-storage/src/main/java/io/druid/storage/hdfs/HdfsDataSegmentPuller.java similarity index 100% rename from hdfs-storage/src/main/java/io/druid/storage/hdfs/HdfsDataSegmentPuller.java rename to extensions/hdfs-storage/src/main/java/io/druid/storage/hdfs/HdfsDataSegmentPuller.java diff --git a/hdfs-storage/src/main/java/io/druid/storage/hdfs/HdfsDataSegmentPusher.java b/extensions/hdfs-storage/src/main/java/io/druid/storage/hdfs/HdfsDataSegmentPusher.java similarity index 100% rename from hdfs-storage/src/main/java/io/druid/storage/hdfs/HdfsDataSegmentPusher.java rename to extensions/hdfs-storage/src/main/java/io/druid/storage/hdfs/HdfsDataSegmentPusher.java diff --git a/hdfs-storage/src/main/java/io/druid/storage/hdfs/HdfsDataSegmentPusherConfig.java b/extensions/hdfs-storage/src/main/java/io/druid/storage/hdfs/HdfsDataSegmentPusherConfig.java similarity index 100% rename from hdfs-storage/src/main/java/io/druid/storage/hdfs/HdfsDataSegmentPusherConfig.java rename to extensions/hdfs-storage/src/main/java/io/druid/storage/hdfs/HdfsDataSegmentPusherConfig.java diff --git a/hdfs-storage/src/main/java/io/druid/storage/hdfs/HdfsStorageDruidModule.java b/extensions/hdfs-storage/src/main/java/io/druid/storage/hdfs/HdfsStorageDruidModule.java similarity index 100% rename from hdfs-storage/src/main/java/io/druid/storage/hdfs/HdfsStorageDruidModule.java rename to extensions/hdfs-storage/src/main/java/io/druid/storage/hdfs/HdfsStorageDruidModule.java diff --git a/hdfs-storage/src/main/java/io/druid/storage/hdfs/tasklog/HdfsTaskLogs.java b/extensions/hdfs-storage/src/main/java/io/druid/storage/hdfs/tasklog/HdfsTaskLogs.java similarity index 100% rename from hdfs-storage/src/main/java/io/druid/storage/hdfs/tasklog/HdfsTaskLogs.java rename to extensions/hdfs-storage/src/main/java/io/druid/storage/hdfs/tasklog/HdfsTaskLogs.java diff --git a/hdfs-storage/src/main/java/io/druid/storage/hdfs/tasklog/HdfsTaskLogsConfig.java b/extensions/hdfs-storage/src/main/java/io/druid/storage/hdfs/tasklog/HdfsTaskLogsConfig.java similarity index 100% rename from hdfs-storage/src/main/java/io/druid/storage/hdfs/tasklog/HdfsTaskLogsConfig.java rename to extensions/hdfs-storage/src/main/java/io/druid/storage/hdfs/tasklog/HdfsTaskLogsConfig.java diff --git a/hdfs-storage/src/main/resources/META-INF/services/io.druid.initialization.DruidModule b/extensions/hdfs-storage/src/main/resources/META-INF/services/io.druid.initialization.DruidModule similarity index 100% rename from hdfs-storage/src/main/resources/META-INF/services/io.druid.initialization.DruidModule rename to extensions/hdfs-storage/src/main/resources/META-INF/services/io.druid.initialization.DruidModule diff --git a/hdfs-storage/src/test/java/io/druid/indexing/common/tasklogs/HdfsTaskLogsTest.java b/extensions/hdfs-storage/src/test/java/io/druid/indexing/common/tasklogs/HdfsTaskLogsTest.java similarity index 100% rename from hdfs-storage/src/test/java/io/druid/indexing/common/tasklogs/HdfsTaskLogsTest.java rename to extensions/hdfs-storage/src/test/java/io/druid/indexing/common/tasklogs/HdfsTaskLogsTest.java diff --git a/histogram/pom.xml b/extensions/histogram/pom.xml similarity index 100% rename from histogram/pom.xml rename to extensions/histogram/pom.xml diff --git a/histogram/src/main/java/io/druid/query/aggregation/histogram/ApproximateHistogram.java b/extensions/histogram/src/main/java/io/druid/query/aggregation/histogram/ApproximateHistogram.java similarity index 100% rename from histogram/src/main/java/io/druid/query/aggregation/histogram/ApproximateHistogram.java rename to extensions/histogram/src/main/java/io/druid/query/aggregation/histogram/ApproximateHistogram.java diff --git a/histogram/src/main/java/io/druid/query/aggregation/histogram/ApproximateHistogramAggregator.java b/extensions/histogram/src/main/java/io/druid/query/aggregation/histogram/ApproximateHistogramAggregator.java similarity index 100% rename from histogram/src/main/java/io/druid/query/aggregation/histogram/ApproximateHistogramAggregator.java rename to extensions/histogram/src/main/java/io/druid/query/aggregation/histogram/ApproximateHistogramAggregator.java diff --git a/histogram/src/main/java/io/druid/query/aggregation/histogram/ApproximateHistogramAggregatorFactory.java b/extensions/histogram/src/main/java/io/druid/query/aggregation/histogram/ApproximateHistogramAggregatorFactory.java similarity index 100% rename from histogram/src/main/java/io/druid/query/aggregation/histogram/ApproximateHistogramAggregatorFactory.java rename to extensions/histogram/src/main/java/io/druid/query/aggregation/histogram/ApproximateHistogramAggregatorFactory.java diff --git a/histogram/src/main/java/io/druid/query/aggregation/histogram/ApproximateHistogramBufferAggregator.java b/extensions/histogram/src/main/java/io/druid/query/aggregation/histogram/ApproximateHistogramBufferAggregator.java similarity index 100% rename from histogram/src/main/java/io/druid/query/aggregation/histogram/ApproximateHistogramBufferAggregator.java rename to extensions/histogram/src/main/java/io/druid/query/aggregation/histogram/ApproximateHistogramBufferAggregator.java diff --git a/histogram/src/main/java/io/druid/query/aggregation/histogram/ApproximateHistogramDruidModule.java b/extensions/histogram/src/main/java/io/druid/query/aggregation/histogram/ApproximateHistogramDruidModule.java similarity index 100% rename from histogram/src/main/java/io/druid/query/aggregation/histogram/ApproximateHistogramDruidModule.java rename to extensions/histogram/src/main/java/io/druid/query/aggregation/histogram/ApproximateHistogramDruidModule.java diff --git a/histogram/src/main/java/io/druid/query/aggregation/histogram/ApproximateHistogramFoldingAggregator.java b/extensions/histogram/src/main/java/io/druid/query/aggregation/histogram/ApproximateHistogramFoldingAggregator.java similarity index 100% rename from histogram/src/main/java/io/druid/query/aggregation/histogram/ApproximateHistogramFoldingAggregator.java rename to extensions/histogram/src/main/java/io/druid/query/aggregation/histogram/ApproximateHistogramFoldingAggregator.java diff --git a/histogram/src/main/java/io/druid/query/aggregation/histogram/ApproximateHistogramFoldingAggregatorFactory.java b/extensions/histogram/src/main/java/io/druid/query/aggregation/histogram/ApproximateHistogramFoldingAggregatorFactory.java similarity index 100% rename from histogram/src/main/java/io/druid/query/aggregation/histogram/ApproximateHistogramFoldingAggregatorFactory.java rename to extensions/histogram/src/main/java/io/druid/query/aggregation/histogram/ApproximateHistogramFoldingAggregatorFactory.java diff --git a/histogram/src/main/java/io/druid/query/aggregation/histogram/ApproximateHistogramFoldingBufferAggregator.java b/extensions/histogram/src/main/java/io/druid/query/aggregation/histogram/ApproximateHistogramFoldingBufferAggregator.java similarity index 100% rename from histogram/src/main/java/io/druid/query/aggregation/histogram/ApproximateHistogramFoldingBufferAggregator.java rename to extensions/histogram/src/main/java/io/druid/query/aggregation/histogram/ApproximateHistogramFoldingBufferAggregator.java diff --git a/histogram/src/main/java/io/druid/query/aggregation/histogram/ApproximateHistogramFoldingSerde.java b/extensions/histogram/src/main/java/io/druid/query/aggregation/histogram/ApproximateHistogramFoldingSerde.java similarity index 100% rename from histogram/src/main/java/io/druid/query/aggregation/histogram/ApproximateHistogramFoldingSerde.java rename to extensions/histogram/src/main/java/io/druid/query/aggregation/histogram/ApproximateHistogramFoldingSerde.java diff --git a/histogram/src/main/java/io/druid/query/aggregation/histogram/ApproximateHistogramPostAggregator.java b/extensions/histogram/src/main/java/io/druid/query/aggregation/histogram/ApproximateHistogramPostAggregator.java similarity index 100% rename from histogram/src/main/java/io/druid/query/aggregation/histogram/ApproximateHistogramPostAggregator.java rename to extensions/histogram/src/main/java/io/druid/query/aggregation/histogram/ApproximateHistogramPostAggregator.java diff --git a/histogram/src/main/java/io/druid/query/aggregation/histogram/ArrayUtils.java b/extensions/histogram/src/main/java/io/druid/query/aggregation/histogram/ArrayUtils.java similarity index 100% rename from histogram/src/main/java/io/druid/query/aggregation/histogram/ArrayUtils.java rename to extensions/histogram/src/main/java/io/druid/query/aggregation/histogram/ArrayUtils.java diff --git a/histogram/src/main/java/io/druid/query/aggregation/histogram/BucketsPostAggregator.java b/extensions/histogram/src/main/java/io/druid/query/aggregation/histogram/BucketsPostAggregator.java similarity index 100% rename from histogram/src/main/java/io/druid/query/aggregation/histogram/BucketsPostAggregator.java rename to extensions/histogram/src/main/java/io/druid/query/aggregation/histogram/BucketsPostAggregator.java diff --git a/histogram/src/main/java/io/druid/query/aggregation/histogram/BufferUtils.java b/extensions/histogram/src/main/java/io/druid/query/aggregation/histogram/BufferUtils.java similarity index 100% rename from histogram/src/main/java/io/druid/query/aggregation/histogram/BufferUtils.java rename to extensions/histogram/src/main/java/io/druid/query/aggregation/histogram/BufferUtils.java diff --git a/histogram/src/main/java/io/druid/query/aggregation/histogram/CustomBucketsPostAggregator.java b/extensions/histogram/src/main/java/io/druid/query/aggregation/histogram/CustomBucketsPostAggregator.java similarity index 100% rename from histogram/src/main/java/io/druid/query/aggregation/histogram/CustomBucketsPostAggregator.java rename to extensions/histogram/src/main/java/io/druid/query/aggregation/histogram/CustomBucketsPostAggregator.java diff --git a/histogram/src/main/java/io/druid/query/aggregation/histogram/EqualBucketsPostAggregator.java b/extensions/histogram/src/main/java/io/druid/query/aggregation/histogram/EqualBucketsPostAggregator.java similarity index 100% rename from histogram/src/main/java/io/druid/query/aggregation/histogram/EqualBucketsPostAggregator.java rename to extensions/histogram/src/main/java/io/druid/query/aggregation/histogram/EqualBucketsPostAggregator.java diff --git a/histogram/src/main/java/io/druid/query/aggregation/histogram/Histogram.java b/extensions/histogram/src/main/java/io/druid/query/aggregation/histogram/Histogram.java similarity index 100% rename from histogram/src/main/java/io/druid/query/aggregation/histogram/Histogram.java rename to extensions/histogram/src/main/java/io/druid/query/aggregation/histogram/Histogram.java diff --git a/histogram/src/main/java/io/druid/query/aggregation/histogram/MaxPostAggregator.java b/extensions/histogram/src/main/java/io/druid/query/aggregation/histogram/MaxPostAggregator.java similarity index 100% rename from histogram/src/main/java/io/druid/query/aggregation/histogram/MaxPostAggregator.java rename to extensions/histogram/src/main/java/io/druid/query/aggregation/histogram/MaxPostAggregator.java diff --git a/histogram/src/main/java/io/druid/query/aggregation/histogram/MinPostAggregator.java b/extensions/histogram/src/main/java/io/druid/query/aggregation/histogram/MinPostAggregator.java similarity index 100% rename from histogram/src/main/java/io/druid/query/aggregation/histogram/MinPostAggregator.java rename to extensions/histogram/src/main/java/io/druid/query/aggregation/histogram/MinPostAggregator.java diff --git a/histogram/src/main/java/io/druid/query/aggregation/histogram/QuantilePostAggregator.java b/extensions/histogram/src/main/java/io/druid/query/aggregation/histogram/QuantilePostAggregator.java similarity index 100% rename from histogram/src/main/java/io/druid/query/aggregation/histogram/QuantilePostAggregator.java rename to extensions/histogram/src/main/java/io/druid/query/aggregation/histogram/QuantilePostAggregator.java diff --git a/histogram/src/main/java/io/druid/query/aggregation/histogram/Quantiles.java b/extensions/histogram/src/main/java/io/druid/query/aggregation/histogram/Quantiles.java similarity index 100% rename from histogram/src/main/java/io/druid/query/aggregation/histogram/Quantiles.java rename to extensions/histogram/src/main/java/io/druid/query/aggregation/histogram/Quantiles.java diff --git a/histogram/src/main/java/io/druid/query/aggregation/histogram/QuantilesPostAggregator.java b/extensions/histogram/src/main/java/io/druid/query/aggregation/histogram/QuantilesPostAggregator.java similarity index 100% rename from histogram/src/main/java/io/druid/query/aggregation/histogram/QuantilesPostAggregator.java rename to extensions/histogram/src/main/java/io/druid/query/aggregation/histogram/QuantilesPostAggregator.java diff --git a/histogram/src/main/resources/META-INF/services/io.druid.initialization.DruidModule b/extensions/histogram/src/main/resources/META-INF/services/io.druid.initialization.DruidModule similarity index 100% rename from histogram/src/main/resources/META-INF/services/io.druid.initialization.DruidModule rename to extensions/histogram/src/main/resources/META-INF/services/io.druid.initialization.DruidModule diff --git a/histogram/src/test/java/io/druid/query/aggregation/histogram/ApproximateHistogramAggregatorTest.java b/extensions/histogram/src/test/java/io/druid/query/aggregation/histogram/ApproximateHistogramAggregatorTest.java similarity index 100% rename from histogram/src/test/java/io/druid/query/aggregation/histogram/ApproximateHistogramAggregatorTest.java rename to extensions/histogram/src/test/java/io/druid/query/aggregation/histogram/ApproximateHistogramAggregatorTest.java diff --git a/histogram/src/test/java/io/druid/query/aggregation/histogram/ApproximateHistogramErrorBenchmark.java b/extensions/histogram/src/test/java/io/druid/query/aggregation/histogram/ApproximateHistogramErrorBenchmark.java similarity index 100% rename from histogram/src/test/java/io/druid/query/aggregation/histogram/ApproximateHistogramErrorBenchmark.java rename to extensions/histogram/src/test/java/io/druid/query/aggregation/histogram/ApproximateHistogramErrorBenchmark.java diff --git a/histogram/src/test/java/io/druid/query/aggregation/histogram/ApproximateHistogramGroupByQueryTest.java b/extensions/histogram/src/test/java/io/druid/query/aggregation/histogram/ApproximateHistogramGroupByQueryTest.java similarity index 100% rename from histogram/src/test/java/io/druid/query/aggregation/histogram/ApproximateHistogramGroupByQueryTest.java rename to extensions/histogram/src/test/java/io/druid/query/aggregation/histogram/ApproximateHistogramGroupByQueryTest.java diff --git a/histogram/src/test/java/io/druid/query/aggregation/histogram/ApproximateHistogramPostAggregatorTest.java b/extensions/histogram/src/test/java/io/druid/query/aggregation/histogram/ApproximateHistogramPostAggregatorTest.java similarity index 100% rename from histogram/src/test/java/io/druid/query/aggregation/histogram/ApproximateHistogramPostAggregatorTest.java rename to extensions/histogram/src/test/java/io/druid/query/aggregation/histogram/ApproximateHistogramPostAggregatorTest.java diff --git a/histogram/src/test/java/io/druid/query/aggregation/histogram/ApproximateHistogramTest.java b/extensions/histogram/src/test/java/io/druid/query/aggregation/histogram/ApproximateHistogramTest.java similarity index 100% rename from histogram/src/test/java/io/druid/query/aggregation/histogram/ApproximateHistogramTest.java rename to extensions/histogram/src/test/java/io/druid/query/aggregation/histogram/ApproximateHistogramTest.java diff --git a/histogram/src/test/java/io/druid/query/aggregation/histogram/ApproximateHistogramTopNQueryTest.java b/extensions/histogram/src/test/java/io/druid/query/aggregation/histogram/ApproximateHistogramTopNQueryTest.java similarity index 100% rename from histogram/src/test/java/io/druid/query/aggregation/histogram/ApproximateHistogramTopNQueryTest.java rename to extensions/histogram/src/test/java/io/druid/query/aggregation/histogram/ApproximateHistogramTopNQueryTest.java diff --git a/histogram/src/test/java/io/druid/query/aggregation/histogram/QuantilesTest.java b/extensions/histogram/src/test/java/io/druid/query/aggregation/histogram/QuantilesTest.java similarity index 100% rename from histogram/src/test/java/io/druid/query/aggregation/histogram/QuantilesTest.java rename to extensions/histogram/src/test/java/io/druid/query/aggregation/histogram/QuantilesTest.java diff --git a/kafka-eight/pom.xml b/extensions/kafka-eight/pom.xml similarity index 100% rename from kafka-eight/pom.xml rename to extensions/kafka-eight/pom.xml diff --git a/kafka-eight/src/main/java/io/druid/firehose/kafka/KafkaEightDruidModule.java b/extensions/kafka-eight/src/main/java/io/druid/firehose/kafka/KafkaEightDruidModule.java similarity index 100% rename from kafka-eight/src/main/java/io/druid/firehose/kafka/KafkaEightDruidModule.java rename to extensions/kafka-eight/src/main/java/io/druid/firehose/kafka/KafkaEightDruidModule.java diff --git a/kafka-eight/src/main/java/io/druid/firehose/kafka/KafkaEightFirehoseFactory.java b/extensions/kafka-eight/src/main/java/io/druid/firehose/kafka/KafkaEightFirehoseFactory.java similarity index 100% rename from kafka-eight/src/main/java/io/druid/firehose/kafka/KafkaEightFirehoseFactory.java rename to extensions/kafka-eight/src/main/java/io/druid/firehose/kafka/KafkaEightFirehoseFactory.java diff --git a/kafka-eight/src/main/resources/META-INF/services/io.druid.initialization.DruidModule b/extensions/kafka-eight/src/main/resources/META-INF/services/io.druid.initialization.DruidModule similarity index 100% rename from kafka-eight/src/main/resources/META-INF/services/io.druid.initialization.DruidModule rename to extensions/kafka-eight/src/main/resources/META-INF/services/io.druid.initialization.DruidModule diff --git a/kafka-seven/pom.xml b/extensions/kafka-seven/pom.xml similarity index 100% rename from kafka-seven/pom.xml rename to extensions/kafka-seven/pom.xml diff --git a/kafka-seven/src/main/java/io/druid/firehose/kafka/KafkaSevenDruidModule.java b/extensions/kafka-seven/src/main/java/io/druid/firehose/kafka/KafkaSevenDruidModule.java similarity index 100% rename from kafka-seven/src/main/java/io/druid/firehose/kafka/KafkaSevenDruidModule.java rename to extensions/kafka-seven/src/main/java/io/druid/firehose/kafka/KafkaSevenDruidModule.java diff --git a/kafka-seven/src/main/java/io/druid/firehose/kafka/KafkaSevenFirehoseFactory.java b/extensions/kafka-seven/src/main/java/io/druid/firehose/kafka/KafkaSevenFirehoseFactory.java similarity index 100% rename from kafka-seven/src/main/java/io/druid/firehose/kafka/KafkaSevenFirehoseFactory.java rename to extensions/kafka-seven/src/main/java/io/druid/firehose/kafka/KafkaSevenFirehoseFactory.java diff --git a/kafka-seven/src/main/resources/META-INF/services/io.druid.initialization.DruidModule b/extensions/kafka-seven/src/main/resources/META-INF/services/io.druid.initialization.DruidModule similarity index 100% rename from kafka-seven/src/main/resources/META-INF/services/io.druid.initialization.DruidModule rename to extensions/kafka-seven/src/main/resources/META-INF/services/io.druid.initialization.DruidModule diff --git a/mysql-metadata-storage/pom.xml b/extensions/mysql-metadata-storage/pom.xml similarity index 100% rename from mysql-metadata-storage/pom.xml rename to extensions/mysql-metadata-storage/pom.xml diff --git a/mysql-metadata-storage/src/main/java/io/druid/metadata/storage/mysql/MySQLConnector.java b/extensions/mysql-metadata-storage/src/main/java/io/druid/metadata/storage/mysql/MySQLConnector.java similarity index 100% rename from mysql-metadata-storage/src/main/java/io/druid/metadata/storage/mysql/MySQLConnector.java rename to extensions/mysql-metadata-storage/src/main/java/io/druid/metadata/storage/mysql/MySQLConnector.java diff --git a/mysql-metadata-storage/src/main/java/io/druid/metadata/storage/mysql/MySQLMetadataStorageModule.java b/extensions/mysql-metadata-storage/src/main/java/io/druid/metadata/storage/mysql/MySQLMetadataStorageModule.java similarity index 100% rename from mysql-metadata-storage/src/main/java/io/druid/metadata/storage/mysql/MySQLMetadataStorageModule.java rename to extensions/mysql-metadata-storage/src/main/java/io/druid/metadata/storage/mysql/MySQLMetadataStorageModule.java diff --git a/mysql-metadata-storage/src/main/resources/META-INF/services/io.druid.initialization.DruidModule b/extensions/mysql-metadata-storage/src/main/resources/META-INF/services/io.druid.initialization.DruidModule similarity index 100% rename from mysql-metadata-storage/src/main/resources/META-INF/services/io.druid.initialization.DruidModule rename to extensions/mysql-metadata-storage/src/main/resources/META-INF/services/io.druid.initialization.DruidModule diff --git a/postgresql-metadata-storage/pom.xml b/extensions/postgresql-metadata-storage/pom.xml similarity index 100% rename from postgresql-metadata-storage/pom.xml rename to extensions/postgresql-metadata-storage/pom.xml diff --git a/postgresql-metadata-storage/src/main/java/io/druid/metadata/storage/postgresql/PostgreSQLConnector.java b/extensions/postgresql-metadata-storage/src/main/java/io/druid/metadata/storage/postgresql/PostgreSQLConnector.java similarity index 100% rename from postgresql-metadata-storage/src/main/java/io/druid/metadata/storage/postgresql/PostgreSQLConnector.java rename to extensions/postgresql-metadata-storage/src/main/java/io/druid/metadata/storage/postgresql/PostgreSQLConnector.java diff --git a/postgresql-metadata-storage/src/main/java/io/druid/metadata/storage/postgresql/PostgreSQLMetadataStorageModule.java b/extensions/postgresql-metadata-storage/src/main/java/io/druid/metadata/storage/postgresql/PostgreSQLMetadataStorageModule.java similarity index 100% rename from postgresql-metadata-storage/src/main/java/io/druid/metadata/storage/postgresql/PostgreSQLMetadataStorageModule.java rename to extensions/postgresql-metadata-storage/src/main/java/io/druid/metadata/storage/postgresql/PostgreSQLMetadataStorageModule.java diff --git a/postgresql-metadata-storage/src/main/resources/META-INF/services/io.druid.initialization.DruidModule b/extensions/postgresql-metadata-storage/src/main/resources/META-INF/services/io.druid.initialization.DruidModule similarity index 100% rename from postgresql-metadata-storage/src/main/resources/META-INF/services/io.druid.initialization.DruidModule rename to extensions/postgresql-metadata-storage/src/main/resources/META-INF/services/io.druid.initialization.DruidModule diff --git a/rabbitmq/pom.xml b/extensions/rabbitmq/pom.xml similarity index 100% rename from rabbitmq/pom.xml rename to extensions/rabbitmq/pom.xml diff --git a/rabbitmq/src/main/java/io/druid/firehose/rabbitmq/JacksonifiedConnectionFactory.java b/extensions/rabbitmq/src/main/java/io/druid/firehose/rabbitmq/JacksonifiedConnectionFactory.java similarity index 100% rename from rabbitmq/src/main/java/io/druid/firehose/rabbitmq/JacksonifiedConnectionFactory.java rename to extensions/rabbitmq/src/main/java/io/druid/firehose/rabbitmq/JacksonifiedConnectionFactory.java diff --git a/rabbitmq/src/main/java/io/druid/firehose/rabbitmq/RabbitMQDruidModule.java b/extensions/rabbitmq/src/main/java/io/druid/firehose/rabbitmq/RabbitMQDruidModule.java similarity index 100% rename from rabbitmq/src/main/java/io/druid/firehose/rabbitmq/RabbitMQDruidModule.java rename to extensions/rabbitmq/src/main/java/io/druid/firehose/rabbitmq/RabbitMQDruidModule.java diff --git a/rabbitmq/src/main/java/io/druid/firehose/rabbitmq/RabbitMQFirehoseConfig.java b/extensions/rabbitmq/src/main/java/io/druid/firehose/rabbitmq/RabbitMQFirehoseConfig.java similarity index 100% rename from rabbitmq/src/main/java/io/druid/firehose/rabbitmq/RabbitMQFirehoseConfig.java rename to extensions/rabbitmq/src/main/java/io/druid/firehose/rabbitmq/RabbitMQFirehoseConfig.java diff --git a/rabbitmq/src/main/java/io/druid/firehose/rabbitmq/RabbitMQFirehoseFactory.java b/extensions/rabbitmq/src/main/java/io/druid/firehose/rabbitmq/RabbitMQFirehoseFactory.java similarity index 100% rename from rabbitmq/src/main/java/io/druid/firehose/rabbitmq/RabbitMQFirehoseFactory.java rename to extensions/rabbitmq/src/main/java/io/druid/firehose/rabbitmq/RabbitMQFirehoseFactory.java diff --git a/rabbitmq/src/main/resources/META-INF/services/io.druid.initialization.DruidModule b/extensions/rabbitmq/src/main/resources/META-INF/services/io.druid.initialization.DruidModule similarity index 100% rename from rabbitmq/src/main/resources/META-INF/services/io.druid.initialization.DruidModule rename to extensions/rabbitmq/src/main/resources/META-INF/services/io.druid.initialization.DruidModule diff --git a/rabbitmq/src/test/java/io/druid/examples/rabbitmq/RabbitMQFirehoseFactoryTest.java b/extensions/rabbitmq/src/test/java/io/druid/examples/rabbitmq/RabbitMQFirehoseFactoryTest.java similarity index 100% rename from rabbitmq/src/test/java/io/druid/examples/rabbitmq/RabbitMQFirehoseFactoryTest.java rename to extensions/rabbitmq/src/test/java/io/druid/examples/rabbitmq/RabbitMQFirehoseFactoryTest.java diff --git a/rabbitmq/src/test/java/io/druid/examples/rabbitmq/RabbitMQProducerMain.java b/extensions/rabbitmq/src/test/java/io/druid/examples/rabbitmq/RabbitMQProducerMain.java similarity index 100% rename from rabbitmq/src/test/java/io/druid/examples/rabbitmq/RabbitMQProducerMain.java rename to extensions/rabbitmq/src/test/java/io/druid/examples/rabbitmq/RabbitMQProducerMain.java diff --git a/s3-extensions/pom.xml b/extensions/s3-extensions/pom.xml similarity index 100% rename from s3-extensions/pom.xml rename to extensions/s3-extensions/pom.xml diff --git a/s3-extensions/src/main/java/io/druid/firehose/s3/S3FirehoseDruidModule.java b/extensions/s3-extensions/src/main/java/io/druid/firehose/s3/S3FirehoseDruidModule.java similarity index 100% rename from s3-extensions/src/main/java/io/druid/firehose/s3/S3FirehoseDruidModule.java rename to extensions/s3-extensions/src/main/java/io/druid/firehose/s3/S3FirehoseDruidModule.java diff --git a/s3-extensions/src/main/java/io/druid/firehose/s3/StaticS3FirehoseFactory.java b/extensions/s3-extensions/src/main/java/io/druid/firehose/s3/StaticS3FirehoseFactory.java similarity index 100% rename from s3-extensions/src/main/java/io/druid/firehose/s3/StaticS3FirehoseFactory.java rename to extensions/s3-extensions/src/main/java/io/druid/firehose/s3/StaticS3FirehoseFactory.java diff --git a/s3-extensions/src/main/java/io/druid/storage/s3/AWSCredentialsConfig.java b/extensions/s3-extensions/src/main/java/io/druid/storage/s3/AWSCredentialsConfig.java similarity index 100% rename from s3-extensions/src/main/java/io/druid/storage/s3/AWSCredentialsConfig.java rename to extensions/s3-extensions/src/main/java/io/druid/storage/s3/AWSCredentialsConfig.java diff --git a/s3-extensions/src/main/java/io/druid/storage/s3/AWSSessionCredentialsAdapter.java b/extensions/s3-extensions/src/main/java/io/druid/storage/s3/AWSSessionCredentialsAdapter.java similarity index 100% rename from s3-extensions/src/main/java/io/druid/storage/s3/AWSSessionCredentialsAdapter.java rename to extensions/s3-extensions/src/main/java/io/druid/storage/s3/AWSSessionCredentialsAdapter.java diff --git a/s3-extensions/src/main/java/io/druid/storage/s3/FileSessionCredentialsProvider.java b/extensions/s3-extensions/src/main/java/io/druid/storage/s3/FileSessionCredentialsProvider.java similarity index 100% rename from s3-extensions/src/main/java/io/druid/storage/s3/FileSessionCredentialsProvider.java rename to extensions/s3-extensions/src/main/java/io/druid/storage/s3/FileSessionCredentialsProvider.java diff --git a/s3-extensions/src/main/java/io/druid/storage/s3/S3DataSegmentArchiver.java b/extensions/s3-extensions/src/main/java/io/druid/storage/s3/S3DataSegmentArchiver.java similarity index 100% rename from s3-extensions/src/main/java/io/druid/storage/s3/S3DataSegmentArchiver.java rename to extensions/s3-extensions/src/main/java/io/druid/storage/s3/S3DataSegmentArchiver.java diff --git a/s3-extensions/src/main/java/io/druid/storage/s3/S3DataSegmentArchiverConfig.java b/extensions/s3-extensions/src/main/java/io/druid/storage/s3/S3DataSegmentArchiverConfig.java similarity index 100% rename from s3-extensions/src/main/java/io/druid/storage/s3/S3DataSegmentArchiverConfig.java rename to extensions/s3-extensions/src/main/java/io/druid/storage/s3/S3DataSegmentArchiverConfig.java diff --git a/s3-extensions/src/main/java/io/druid/storage/s3/S3DataSegmentKiller.java b/extensions/s3-extensions/src/main/java/io/druid/storage/s3/S3DataSegmentKiller.java similarity index 100% rename from s3-extensions/src/main/java/io/druid/storage/s3/S3DataSegmentKiller.java rename to extensions/s3-extensions/src/main/java/io/druid/storage/s3/S3DataSegmentKiller.java diff --git a/s3-extensions/src/main/java/io/druid/storage/s3/S3DataSegmentMover.java b/extensions/s3-extensions/src/main/java/io/druid/storage/s3/S3DataSegmentMover.java similarity index 100% rename from s3-extensions/src/main/java/io/druid/storage/s3/S3DataSegmentMover.java rename to extensions/s3-extensions/src/main/java/io/druid/storage/s3/S3DataSegmentMover.java diff --git a/s3-extensions/src/main/java/io/druid/storage/s3/S3DataSegmentPuller.java b/extensions/s3-extensions/src/main/java/io/druid/storage/s3/S3DataSegmentPuller.java similarity index 100% rename from s3-extensions/src/main/java/io/druid/storage/s3/S3DataSegmentPuller.java rename to extensions/s3-extensions/src/main/java/io/druid/storage/s3/S3DataSegmentPuller.java diff --git a/s3-extensions/src/main/java/io/druid/storage/s3/S3DataSegmentPusher.java b/extensions/s3-extensions/src/main/java/io/druid/storage/s3/S3DataSegmentPusher.java similarity index 100% rename from s3-extensions/src/main/java/io/druid/storage/s3/S3DataSegmentPusher.java rename to extensions/s3-extensions/src/main/java/io/druid/storage/s3/S3DataSegmentPusher.java diff --git a/s3-extensions/src/main/java/io/druid/storage/s3/S3DataSegmentPusherConfig.java b/extensions/s3-extensions/src/main/java/io/druid/storage/s3/S3DataSegmentPusherConfig.java similarity index 100% rename from s3-extensions/src/main/java/io/druid/storage/s3/S3DataSegmentPusherConfig.java rename to extensions/s3-extensions/src/main/java/io/druid/storage/s3/S3DataSegmentPusherConfig.java diff --git a/s3-extensions/src/main/java/io/druid/storage/s3/S3StorageDruidModule.java b/extensions/s3-extensions/src/main/java/io/druid/storage/s3/S3StorageDruidModule.java similarity index 100% rename from s3-extensions/src/main/java/io/druid/storage/s3/S3StorageDruidModule.java rename to extensions/s3-extensions/src/main/java/io/druid/storage/s3/S3StorageDruidModule.java diff --git a/s3-extensions/src/main/java/io/druid/storage/s3/S3TaskLogs.java b/extensions/s3-extensions/src/main/java/io/druid/storage/s3/S3TaskLogs.java similarity index 100% rename from s3-extensions/src/main/java/io/druid/storage/s3/S3TaskLogs.java rename to extensions/s3-extensions/src/main/java/io/druid/storage/s3/S3TaskLogs.java diff --git a/s3-extensions/src/main/java/io/druid/storage/s3/S3TaskLogsConfig.java b/extensions/s3-extensions/src/main/java/io/druid/storage/s3/S3TaskLogsConfig.java similarity index 100% rename from s3-extensions/src/main/java/io/druid/storage/s3/S3TaskLogsConfig.java rename to extensions/s3-extensions/src/main/java/io/druid/storage/s3/S3TaskLogsConfig.java diff --git a/s3-extensions/src/main/java/io/druid/storage/s3/S3Utils.java b/extensions/s3-extensions/src/main/java/io/druid/storage/s3/S3Utils.java similarity index 100% rename from s3-extensions/src/main/java/io/druid/storage/s3/S3Utils.java rename to extensions/s3-extensions/src/main/java/io/druid/storage/s3/S3Utils.java diff --git a/s3-extensions/src/main/resources/META-INF/services/io.druid.initialization.DruidModule b/extensions/s3-extensions/src/main/resources/META-INF/services/io.druid.initialization.DruidModule similarity index 100% rename from s3-extensions/src/main/resources/META-INF/services/io.druid.initialization.DruidModule rename to extensions/s3-extensions/src/main/resources/META-INF/services/io.druid.initialization.DruidModule diff --git a/s3-extensions/src/test/java/io/druid/storage/s3/S3DataSegmentMoverTest.java b/extensions/s3-extensions/src/test/java/io/druid/storage/s3/S3DataSegmentMoverTest.java similarity index 100% rename from s3-extensions/src/test/java/io/druid/storage/s3/S3DataSegmentMoverTest.java rename to extensions/s3-extensions/src/test/java/io/druid/storage/s3/S3DataSegmentMoverTest.java diff --git a/s3-extensions/src/test/java/io/druid/storage/s3/TestAWSCredentialsProvider.java b/extensions/s3-extensions/src/test/java/io/druid/storage/s3/TestAWSCredentialsProvider.java similarity index 100% rename from s3-extensions/src/test/java/io/druid/storage/s3/TestAWSCredentialsProvider.java rename to extensions/s3-extensions/src/test/java/io/druid/storage/s3/TestAWSCredentialsProvider.java diff --git a/s3-extensions/src/test/java/io/druid/storage/s3/TestFileSessionCredentialsProvider.java b/extensions/s3-extensions/src/test/java/io/druid/storage/s3/TestFileSessionCredentialsProvider.java similarity index 100% rename from s3-extensions/src/test/java/io/druid/storage/s3/TestFileSessionCredentialsProvider.java rename to extensions/s3-extensions/src/test/java/io/druid/storage/s3/TestFileSessionCredentialsProvider.java diff --git a/indexing-service/src/main/java/io/druid/indexing/common/task/IndexTask.java b/indexing-service/src/main/java/io/druid/indexing/common/task/IndexTask.java index 00da8cbc1cb..c8474893db8 100644 --- a/indexing-service/src/main/java/io/druid/indexing/common/task/IndexTask.java +++ b/indexing-service/src/main/java/io/druid/indexing/common/task/IndexTask.java @@ -126,7 +126,7 @@ public class IndexTask extends AbstractFixedIntervalTask @JsonCreator public IndexTask( @JsonProperty("id") String id, - @JsonProperty("schema") IndexIngestionSpec ingestionSchema, + @JsonProperty("spec") IndexIngestionSpec ingestionSchema, @JacksonInject ObjectMapper jsonMapper ) { @@ -148,7 +148,7 @@ public class IndexTask extends AbstractFixedIntervalTask return "index"; } - @JsonProperty("schema") + @JsonProperty("spec") public IndexIngestionSpec getIngestionSchema() { return ingestionSchema; diff --git a/install/log4j.xml b/install/log4j.xml deleted file mode 100644 index aa6eb8cc13e..00000000000 --- a/install/log4j.xml +++ /dev/null @@ -1,18 +0,0 @@ - - - - - - - - - - - - - - - - - - diff --git a/install/postgresql-schema.sql b/install/postgresql-schema.sql deleted file mode 100644 index 180ec5e6702..00000000000 --- a/install/postgresql-schema.sql +++ /dev/null @@ -1,49 +0,0 @@ --- Table structure for table `config` --- - - -DROP TABLE IF EXISTS prod_config; -/*!40101 SET @saved_cs_client = @@character_set_client */; -/*!40101 SET character_set_client = utf8 */; -CREATE TABLE prod_config ( - name varchar(255) NOT NULL, - payload bytea NOT NULL, - PRIMARY KEY (name) -); -/*!40101 SET character_set_client = @saved_cs_client */; - --- --- Table structure for table `rules` --- -DROP TABLE IF EXISTS prod_rules; -/*!40101 SET @saved_cs_client = @@character_set_client */; -/*!40101 SET character_set_client = utf8 */; -CREATE TABLE prod_rules ( - id varchar(255) NOT NULL, - dataSource varchar(255) NOT NULL, - version text NOT NULL, - payload text NOT NULL, - PRIMARY KEY (id) -); -/*!40101 SET character_set_client = @saved_cs_client */; - --- --- Table structure for table `segments` --- - -DROP TABLE IF EXISTS prod_segments; -/*!40101 SET @saved_cs_client = @@character_set_client */; -/*!40101 SET character_set_client = utf8 */; -CREATE TABLE prod_segments ( - id varchar(255) NOT NULL, - dataSource varchar(255) NOT NULL, - created_date text NOT NULL, - start text NOT NULL, - "end" text NOT NULL, - partitioned SMALLINT NOT NULL, - version text NOT NULL, - used boolean NOT NULL, - payload text NOT NULL, - PRIMARY KEY (id) -); - diff --git a/CodeStyle.jar b/intellij_formatting.jar similarity index 100% rename from CodeStyle.jar rename to intellij_formatting.jar diff --git a/pom.xml b/pom.xml index 6011e72bf6c..e38caeef5e8 100644 --- a/pom.xml +++ b/pom.xml @@ -54,15 +54,15 @@ server services - cassandra-storage - hdfs-storage - s3-extensions - kafka-seven - kafka-eight - rabbitmq - histogram - mysql-metadata-storage - postgresql-metadata-storage + extensions/cassandra-storage + extensions/hdfs-storage + extensions/s3-extensions + extensions/kafka-seven + extensions/kafka-eight + extensions/rabbitmq + extensions/histogram + extensions/mysql-metadata-storage + extensions/postgresql-metadata-storage diff --git a/processing/src/main/java/io/druid/guice/GuiceInjectors.java b/processing/src/main/java/io/druid/guice/GuiceInjectors.java index 2628e2feee4..9a3425c872f 100644 --- a/processing/src/main/java/io/druid/guice/GuiceInjectors.java +++ b/processing/src/main/java/io/druid/guice/GuiceInjectors.java @@ -38,7 +38,7 @@ public class GuiceInjectors return Guice.createInjector( new DruidGuiceExtensions(), new JacksonModule(), - new PropertiesModule(Arrays.asList("global.runtime.properties", "runtime.properties")), + new PropertiesModule(Arrays.asList("common.runtime.properties", "runtime.properties")), new ConfigModule(), new Module() { @@ -57,7 +57,7 @@ public class GuiceInjectors List theModules = Lists.newArrayList(); theModules.add(new DruidGuiceExtensions()); theModules.add(new JacksonModule()); - theModules.add(new PropertiesModule(Arrays.asList("global.runtime.properties", "runtime.properties"))); + theModules.add(new PropertiesModule(Arrays.asList("common.runtime.properties", "runtime.properties"))); theModules.add(new ConfigModule()); theModules.add( new Module() diff --git a/server/src/main/java/io/druid/guice/ParsersModule.java b/server/src/main/java/io/druid/guice/ParsersModule.java index d4b46c872fa..2915d1dce34 100644 --- a/server/src/main/java/io/druid/guice/ParsersModule.java +++ b/server/src/main/java/io/druid/guice/ParsersModule.java @@ -25,7 +25,7 @@ import com.fasterxml.jackson.databind.module.SimpleModule; import com.google.inject.Binder; import io.druid.data.input.ProtoBufInputRowParser; import io.druid.initialization.DruidModule; -import io.druid.segment.realtime.firehose.IrcParser; +import io.druid.segment.realtime.firehose.IrcInputRowParser; import java.util.Arrays; import java.util.List; @@ -46,7 +46,7 @@ public class ParsersModule implements DruidModule new SimpleModule("ParsersModule") .registerSubtypes( new NamedType(ProtoBufInputRowParser.class, "protobuf"), - new NamedType(IrcParser.class, "irc") + new NamedType(IrcInputRowParser.class, "irc") ) ); } diff --git a/server/src/main/java/io/druid/segment/realtime/firehose/IrcDecoder.java b/server/src/main/java/io/druid/segment/realtime/firehose/IrcDecoder.java index 73872162199..abcd152a621 100644 --- a/server/src/main/java/io/druid/segment/realtime/firehose/IrcDecoder.java +++ b/server/src/main/java/io/druid/segment/realtime/firehose/IrcDecoder.java @@ -26,8 +26,8 @@ import org.joda.time.DateTime; @JsonTypeInfo(use = JsonTypeInfo.Id.NAME, property = "type") @JsonSubTypes({ - @JsonSubTypes.Type(name = "wikipedia", value = WikipediaIrcDecoder.class) - }) + @JsonSubTypes.Type(name = "wikipedia", value = WikipediaIrcDecoder.class) +}) public interface IrcDecoder { public InputRow decodeMessage(DateTime timestamp, String channel, String msg); diff --git a/server/src/main/java/io/druid/segment/realtime/firehose/IrcFirehoseFactory.java b/server/src/main/java/io/druid/segment/realtime/firehose/IrcFirehoseFactory.java index fdb98cf0743..da1f3c0f053 100644 --- a/server/src/main/java/io/druid/segment/realtime/firehose/IrcFirehoseFactory.java +++ b/server/src/main/java/io/druid/segment/realtime/firehose/IrcFirehoseFactory.java @@ -43,43 +43,8 @@ import java.util.UUID; import java.util.concurrent.LinkedBlockingQueue; /** - *

Example Usage

- * - *

Decoder definition: wikipedia-decoder.json

- *
{@code
- *
- * {
- *   "type": "wikipedia",
- *   "namespaces": {
- *     "#en.wikipedia": {
- *       "": "main",
- *       "Category": "category",
- *       "Template talk": "template talk",
- *       "Help talk": "help talk",
- *       "Media": "media",
- *       "MediaWiki talk": "mediawiki talk",
- *       "File talk": "file talk",
- *       "MediaWiki": "mediawiki",
- *       "User": "user",
- *       "File": "file",
- *       "User talk": "user talk",
- *       "Template": "template",
- *       "Help": "help",
- *       "Special": "special",
- *       "Talk": "talk",
- *       "Category talk": "category talk"
- *     }
- *   },
- *   "geoIpDatabase": "path/to/GeoLite2-City.mmdb"
- * }
- * }
- * *

Example code:

*
{@code
- * IrcDecoder wikipediaDecoder = new ObjectMapper().readValue(
- *   new File("wikipedia-decoder.json"),
- *   IrcDecoder.class
- * );
  *
  * IrcFirehoseFactory factory = new IrcFirehoseFactory(
  *     "wiki123",
@@ -89,32 +54,28 @@ import java.util.concurrent.LinkedBlockingQueue;
  *         "#fr.wikipedia",
  *         "#de.wikipedia",
  *         "#ja.wikipedia"
- *     ),
- *     wikipediaDecoder
+ *     )
  * );
  * }
*/ -public class IrcFirehoseFactory implements FirehoseFactory +public class IrcFirehoseFactory implements FirehoseFactory { private static final Logger log = new Logger(IrcFirehoseFactory.class); private final String nick; private final String host; private final List channels; - private final IrcDecoder decoder; @JsonCreator public IrcFirehoseFactory( @JsonProperty("name") String nick, @JsonProperty("host") String host, - @JsonProperty("channels") List channels, - @JsonProperty("decoder") IrcDecoder decoder + @JsonProperty("channels") List channels ) { this.nick = nick; this.host = host; this.channels = channels; - this.decoder = decoder; } @JsonProperty @@ -135,14 +96,8 @@ public class IrcFirehoseFactory implements FirehoseFactory return channels; } - @JsonProperty - public IrcDecoder getDecoder() - { - return decoder; - } - @Override - public Firehose connect(final IrcParser firehoseParser) throws IOException + public Firehose connect(final IrcInputRowParser firehoseParser) throws IOException { final IRCApi irc = new IRCApiImpl(false); final LinkedBlockingQueue> queue = new LinkedBlockingQueue>(); diff --git a/server/src/main/java/io/druid/segment/realtime/firehose/IrcParser.java b/server/src/main/java/io/druid/segment/realtime/firehose/IrcInputRowParser.java similarity index 59% rename from server/src/main/java/io/druid/segment/realtime/firehose/IrcParser.java rename to server/src/main/java/io/druid/segment/realtime/firehose/IrcInputRowParser.java index 57ae95a3e53..06532543a08 100644 --- a/server/src/main/java/io/druid/segment/realtime/firehose/IrcParser.java +++ b/server/src/main/java/io/druid/segment/realtime/firehose/IrcInputRowParser.java @@ -30,15 +30,50 @@ import io.druid.data.input.impl.ParseSpec; import org.joda.time.DateTime; /** + *

Example Usage

+ *

+ *

Decoder definition: wikipedia-decoder.json

+ *
{@code
+ * 

+ * { + * "type": "wikipedia", + * "namespaces": { + * "#en.wikipedia": { + * "": "main", + * "Category": "category", + * "Template talk": "template talk", + * "Help talk": "help talk", + * "Media": "media", + * "MediaWiki talk": "mediawiki talk", + * "File talk": "file talk", + * "MediaWiki": "mediawiki", + * "User": "user", + * "File": "file", + * "User talk": "user talk", + * "Template": "template", + * "Help": "help", + * "Special": "special", + * "Talk": "talk", + * "Category talk": "category talk" + * } + * }, + * "geoIpDatabase": "path/to/GeoLite2-City.mmdb" + * } + * }

*/ -@JsonTypeName("protoBuf") -public class IrcParser implements InputRowParser> +@JsonTypeName("irc") +public class IrcInputRowParser implements InputRowParser> { + private final ParseSpec parseSpec; private final IrcDecoder decoder; @JsonCreator - public IrcParser(@JsonProperty("decoder") IrcDecoder decoder) + public IrcInputRowParser( + @JsonProperty("parseSpec") ParseSpec parseSpec, + @JsonProperty("decoder") IrcDecoder decoder + ) { + this.parseSpec = parseSpec; this.decoder = decoder; } @@ -54,15 +89,16 @@ public class IrcParser implements InputRowParser> return decoder.decodeMessage(msg.lhs, msg.rhs.getChannelName(), msg.rhs.getText()); } + @JsonProperty @Override public ParseSpec getParseSpec() { - return null; + return parseSpec; } @Override public InputRowParser withParseSpec(ParseSpec parseSpec) { - throw new UnsupportedOperationException(); + return new IrcInputRowParser(parseSpec, decoder); } } \ No newline at end of file diff --git a/server/src/main/java/io/druid/segment/realtime/firehose/WikipediaIrcDecoder.java b/server/src/main/java/io/druid/segment/realtime/firehose/WikipediaIrcDecoder.java index dd715058d41..0dadeb0b277 100644 --- a/server/src/main/java/io/druid/segment/realtime/firehose/WikipediaIrcDecoder.java +++ b/server/src/main/java/io/druid/segment/realtime/firehose/WikipediaIrcDecoder.java @@ -76,28 +76,31 @@ class WikipediaIrcDecoder implements IrcDecoder final Map> namespaces; final String geoIpDatabase; - public WikipediaIrcDecoder( Map> namespaces) { + public WikipediaIrcDecoder(Map> namespaces) + { this(namespaces, null); } @JsonCreator - public WikipediaIrcDecoder(@JsonProperty("namespaces") Map> namespaces, - @JsonProperty("geoIpDatabase") String geoIpDatabase) + public WikipediaIrcDecoder( + @JsonProperty("namespaces") Map> namespaces, + @JsonProperty("geoIpDatabase") String geoIpDatabase + ) { - if(namespaces == null) { + if (namespaces == null) { namespaces = Maps.newHashMap(); } this.namespaces = namespaces; this.geoIpDatabase = geoIpDatabase; File geoDb; - if(geoIpDatabase != null) { + if (geoIpDatabase != null) { geoDb = new File(geoIpDatabase); } else { try { String tmpDir = System.getProperty("java.io.tmpdir"); geoDb = new File(tmpDir, this.getClass().getCanonicalName() + ".GeoLite2-City.mmdb"); - if(!geoDb.exists()) { + if (!geoDb.exists()) { log.info("Downloading geo ip database to [%s]", geoDb); FileUtils.copyInputStreamToFile( @@ -107,13 +110,15 @@ class WikipediaIrcDecoder implements IrcDecoder geoDb ); } - } catch(IOException e) { + } + catch (IOException e) { throw new RuntimeException("Unable to download geo ip database [%s]", e); } } try { geoLookup = new DatabaseReader(geoDb); - } catch(IOException e) { + } + catch (IOException e) { throw new RuntimeException("Unable to open geo ip lookup database", e); } } @@ -137,12 +142,12 @@ class WikipediaIrcDecoder implements IrcDecoder final Map metrics = Maps.newHashMap(); Matcher m = pattern.matcher(msg); - if(!m.matches()) { + if (!m.matches()) { throw new IllegalArgumentException("Invalid input format"); } Matcher shortname = shortnamePattern.matcher(channel); - if(shortname.matches()) { + if (shortname.matches()) { dimensions.put("language", shortname.group(1)); } @@ -154,7 +159,7 @@ class WikipediaIrcDecoder implements IrcDecoder String user = m.group(4); Matcher ipMatch = ipPattern.matcher(user); boolean anonymous = ipMatch.matches(); - if(anonymous) { + if (anonymous) { try { final InetAddress ip = InetAddress.getByName(ipMatch.group()); final Omni lookup = geoLookup.omni(ip); @@ -163,11 +168,14 @@ class WikipediaIrcDecoder implements IrcDecoder dimensions.put("country", lookup.getCountry().getName()); dimensions.put("region", lookup.getMostSpecificSubdivision().getName()); dimensions.put("city", lookup.getCity().getName()); - } catch(UnknownHostException e) { + } + catch (UnknownHostException e) { log.error(e, "invalid ip [%s]", ipMatch.group()); - } catch(IOException e) { + } + catch (IOException e) { log.error(e, "error looking up geo ip"); - } catch(GeoIp2Exception e) { + } + catch (GeoIp2Exception e) { log.error(e, "error looking up geo ip"); } } @@ -181,15 +189,14 @@ class WikipediaIrcDecoder implements IrcDecoder dimensions.put("anonymous", Boolean.toString(anonymous)); String[] parts = page.split(":"); - if(parts.length > 1 && !parts[1].startsWith(" ")) { + if (parts.length > 1 && !parts[1].startsWith(" ")) { Map channelNamespaces = namespaces.get(channel); - if(channelNamespaces != null && channelNamespaces.containsKey(parts[0])) { + if (channelNamespaces != null && channelNamespaces.containsKey(parts[0])) { dimensions.put("namespace", channelNamespaces.get(parts[0])); } else { dimensions.put("namespace", "wikipedia"); } - } - else { + } else { dimensions.put("namespace", "article"); } @@ -222,7 +229,7 @@ class WikipediaIrcDecoder implements IrcDecoder public List getDimension(String dimension) { final String value = dimensions.get(dimension); - if(value != null) { + if (value != null) { return ImmutableList.of(value); } else { return ImmutableList.of(); @@ -230,7 +237,8 @@ class WikipediaIrcDecoder implements IrcDecoder } @Override - public Object getRaw(String dimension) { + public Object getRaw(String dimension) + { return dimensions.get(dimension); } diff --git a/services/src/assembly/assembly.xml b/services/src/assembly/assembly.xml index 6ab4cefa7e6..b5fd6a7889f 100644 --- a/services/src/assembly/assembly.xml +++ b/services/src/assembly/assembly.xml @@ -15,11 +15,11 @@ config - ../examples/config/_global + ../examples/config/_common * - config/_global + config/_common ../examples/config/broker diff --git a/services/src/main/java/io/druid/cli/CliRealtimeExample.java b/services/src/main/java/io/druid/cli/CliRealtimeExample.java index fa5e40df923..a9ce071f000 100644 --- a/services/src/main/java/io/druid/cli/CliRealtimeExample.java +++ b/services/src/main/java/io/druid/cli/CliRealtimeExample.java @@ -22,6 +22,7 @@ package io.druid.cli; import com.google.common.collect.ImmutableList; import com.google.inject.Binder; import com.google.inject.Module; +import com.google.inject.name.Names; import com.metamx.common.logger.Logger; import io.airlift.command.Command; import io.druid.client.DruidServer; @@ -63,6 +64,9 @@ public class CliRealtimeExample extends ServerRunnable @Override public void configure(Binder binder) { + binder.bindConstant().annotatedWith(Names.named("serviceName")).to("druid/realtime"); + binder.bindConstant().annotatedWith(Names.named("servicePort")).to(8084); + binder.bind(DataSegmentPusher.class).to(NoopDataSegmentPusher.class).in(LazySingleton.class); binder.bind(DataSegmentAnnouncer.class).to(NoopDataSegmentAnnouncer.class).in(LazySingleton.class); binder.bind(InventoryView.class).to(NoopInventoryView.class).in(LazySingleton.class);