From a34a06e1926e2006807a252a9387dc13b926db70 Mon Sep 17 00:00:00 2001 From: Clint Wylie Date: Fri, 19 Jul 2024 14:37:21 -0700 Subject: [PATCH] remove Firehose and FirehoseFactory (#16758) changes: * removed `Firehose` and `FirehoseFactory` and remaining implementations which were mostly no longer used after #16602 * Moved `IngestSegmentFirehose` which was still used internally by Hadoop ingestion to `DatasourceRecordReader.SegmentReader` * Rename `SQLFirehoseFactoryDatabaseConnector` to `SQLInputSourceDatabaseConnector` and similar renames for sub-classes * Moved anything remaining in a 'firehose' package somewhere else * Clean up docs on firehose stuff --- docs/configuration/extensions.md | 2 +- docs/configuration/index.md | 5 +- .../extensions-contrib/cloudfiles.md | 56 -- .../development/extensions-core/postgresql.md | 2 +- docs/development/overview.md | 13 +- docs/ingestion/native-batch-firehose.md | 316 +-------- docs/operations/metrics.md | 9 - .../migrate-from-firehose-ingestion.md | 4 +- docs/operations/security-overview.md | 2 +- .../k8s/overlord/common/K8sTestUtils.java | 1 - .../DruidPeonClientIntegrationTest.java | 5 - .../taskadapter/K8sTaskAdapterTest.java | 5 - .../MultiContainerTaskAdapterTest.java | 5 - .../SingleContainerTaskAdapterTest.java | 5 - .../MaterializedViewSupervisorSpec.java | 2 +- .../MaterializedViewSupervisorSpecTest.java | 4 +- .../MaterializedViewSupervisorTest.java | 2 +- .../src/main/resources/defaultMetrics.json | 3 - extensions-core/azure-extensions/pom.xml | 4 - .../kinesis/KinesisIndexTaskSerdeTest.java | 4 +- .../indexing/IndexerControllerContext.java | 2 +- .../client/ControllerChatHandler.java | 2 +- .../indexing/client/WorkerChatHandler.java | 4 +- .../druid/msq/test/MSQTestWorkerContext.java | 2 +- .../MySQLInputSourceDatabaseConnector.java} | 10 +- .../mysql/MySQLMetadataStorageModule.java | 4 +- ...ySQLInputSourceDatabaseConnectorTest.java} | 34 +- ...stgresqlInputSourceDatabaseConnector.java} | 12 +- .../PostgreSQLMetadataStorageModule.java | 4 +- ...esqlInputSourceDatabaseConnectorTest.java} | 26 +- .../hadoop/DatasourceRecordReader.java | 193 ++++- .../indexer/BatchDeltaIngestionTest.java | 12 +- ...tasourceRecordReaderSegmentReaderTest.java | 19 +- .../guice/IndexingServiceFirehoseModule.java | 49 -- .../druid/indexing/common/TaskToolbox.java | 2 +- .../indexing/common/TaskToolboxFactory.java | 2 +- .../indexing/common/task/HadoopIndexTask.java | 4 +- .../druid/indexing/common/task/IndexTask.java | 34 +- .../common/task/NativeCompactionRunner.java | 1 - .../druid/indexing/common/task/Task.java | 12 +- ...putSourceSplitParallelIndexTaskRunner.java | 4 - .../parallel/LegacySinglePhaseSubTask.java | 3 - .../batch/parallel/ParallelIndexIOConfig.java | 17 +- .../parallel/ParallelIndexSupervisorTask.java | 9 +- .../PartialDimensionCardinalityTask.java | 3 - .../PartialDimensionDistributionTask.java | 3 - .../PartialHashSegmentGenerateTask.java | 3 - .../PartialRangeSegmentGenerateTask.java | 3 - .../parallel/PartialSegmentGenerateTask.java | 2 +- .../SinglePhaseParallelIndexTaskRunner.java | 4 - .../batch/parallel/SinglePhaseSubTask.java | 5 +- .../IndexTaskInputRowIteratorBuilder.java | 2 - .../indexing/input/DruidInputSource.java | 5 +- .../indexing/input/DruidSegmentReader.java | 2 +- .../WindowedSegmentId.java | 2 +- .../overlord/sampler/InputSourceSampler.java | 4 +- .../SeekableStreamIndexTask.java | 2 +- .../SeekableStreamIndexTaskRunner.java | 2 +- .../indexing/common/TaskToolboxTest.java | 2 +- .../druid/indexing/common/TestFirehose.java | 118 ---- .../druid/indexing/common/TestIndexTask.java | 1 - .../druid/indexing/common/TestUtils.java | 10 +- .../common/task/BatchAppenderatorsTest.java | 2 +- .../ClientCompactionTaskQuerySerdeTest.java | 4 +- .../task/CompactionTaskParallelRunTest.java | 3 +- .../common/task/CompactionTaskRunTest.java | 4 +- .../common/task/CompactionTaskTest.java | 4 +- .../common/task/IndexIngestionSpecTest.java | 2 - .../indexing/common/task/IndexTaskTest.java | 7 +- .../common/task/IngestionTestBase.java | 7 +- .../indexing/common/task/TaskSerdeTest.java | 29 +- ...bstractMultiPhaseParallelIndexingTest.java | 2 - ...stractParallelIndexSupervisorTaskTest.java | 4 +- .../parallel/HashPartitionTaskKillTest.java | 2 - ...aseParallelIndexingWithNullColumnTest.java | 4 - .../ParallelIndexSupervisorTaskKillTest.java | 3 - ...rallelIndexSupervisorTaskResourceTest.java | 2 - .../ParallelIndexSupervisorTaskSerdeTest.java | 1 - .../ParallelIndexSupervisorTaskTest.java | 3 - .../parallel/ParallelIndexTestingFactory.java | 2 +- .../parallel/RangePartitionTaskKillTest.java | 1 - .../SinglePhaseParallelIndexingTest.java | 6 - .../parallel/SinglePhaseSubTaskSpecTest.java | 1 - .../indexing/input/DruidInputSourceTest.java | 1 - .../WindowedSegmentIdTest.java | 2 +- .../overlord/ForkingTaskRunnerTest.java | 2 - .../SingleTaskBackgroundRunnerTest.java | 2 +- .../indexing/overlord/TaskLifecycleTest.java | 45 +- .../indexing/overlord/TaskQueueTest.java | 1 - .../overlord/TestTaskToolboxFactory.java | 2 +- .../overlord/http/OverlordResourceTest.java | 2 +- .../sampler/IndexTaskSamplerSpecTest.java | 5 +- .../SeekableStreamIndexTaskTestBase.java | 2 +- .../indexing/worker/TaskAnnouncementTest.java | 2 +- .../worker/WorkerTaskManagerTest.java | 2 +- .../worker/WorkerTaskMonitorTest.java | 2 +- .../ITBestEffortRollupParallelIndexTest.java | 2 +- .../EventReceiverFirehoseTestClient.java | 181 ----- .../testing/guice/DruidTestModuleFactory.java | 2 - .../ITBestEffortRollupParallelIndexTest.java | 2 +- .../ITPerfectRollupParallelIndexTest.java | 2 +- .../org/apache/druid/data/input/Firehose.java | 97 --- .../druid/data/input/FirehoseFactory.java | 82 --- .../data/input/SegmentsSplitHintSpec.java | 5 +- .../org/apache/druid/segment/Metadata.java | 2 +- .../impl/InputEntityIteratingReaderTest.java | 4 +- .../apache/druid/guice/FirehoseModule.java | 55 -- .../initialization/CoreInjectorBuilder.java | 2 - .../druid/metadata/BasicDataSourceExt.java | 2 +- ...a => SQLInputSourceDatabaseConnector.java} | 2 +- .../druid/metadata/input/SqlEntity.java | 20 +- .../druid/metadata/input/SqlInputSource.java | 23 +- .../rpc/indexing/SpecificTaskRetryPolicy.java | 2 +- .../realtime/{firehose => }/ChatHandler.java | 2 +- .../{firehose => }/ChatHandlerProvider.java | 2 +- .../{firehose => }/ChatHandlerResource.java | 2 +- .../realtime/{firehose => }/ChatHandlers.java | 2 +- .../druid/segment/realtime/FireHydrant.java | 2 + .../NoopChatHandlerProvider.java | 2 +- .../ServiceAnnouncingChatHandlerProvider.java | 2 +- .../WindowedStorageAdapter.java | 2 +- .../appenderator/AppenderatorImpl.java | 4 +- .../appenderator/BatchAppenderator.java | 2 +- .../appenderator/StreamAppenderator.java | 2 +- .../firehose/ClippedFirehoseFactory.java | 81 --- .../realtime/firehose/EventReceiver.java | 28 - .../EventReceiverFirehoseFactory.java | 659 ------------------ .../firehose/FixedCountFirehoseFactory.java | 93 --- .../firehose/IngestSegmentFirehose.java | 210 ------ .../realtime/firehose/PredicateFirehose.java | 89 --- .../firehose/TimedShutoffFirehoseFactory.java | 139 ---- .../realtime/{firehose => }/package-info.java | 2 +- .../jetty/ChatHandlerServerModule.java | 2 +- .../jetty/CliIndexerServerModule.java | 2 +- .../TaskIdResponseHeaderFilterHolder.java | 2 +- .../metrics/EventReceiverFirehoseMetric.java | 48 -- .../metrics/EventReceiverFirehoseMonitor.java | 90 --- .../EventReceiverFirehoseRegister.java | 56 -- .../druid/server/metrics/MetricsModule.java | 1 - .../discovery/ServiceAnnouncerTest.java | 27 +- .../druid/guice/FirehoseModuleTest.java | 93 --- .../druid/metadata/input/SqlEntityTest.java | 4 +- .../metadata/input/SqlInputSourceTest.java | 111 ++- .../druid/metadata/input/SqlTestUtils.java | 55 +- .../ChatHandlerResourceTest.java | 2 +- ...viceAnnouncingChatHandlerProviderTest.java | 2 +- .../EventReceiverFirehoseIdleTest.java | 136 ---- .../firehose/EventReceiverFirehoseTest.java | 442 ------------ .../java/org/apache/druid/cli/CliIndexer.java | 2 - .../apache/druid/cli/CliMiddleManager.java | 6 +- .../org/apache/druid/cli/CliOverlord.java | 6 +- .../java/org/apache/druid/cli/CliPeon.java | 8 +- .../cli/validate/DruidJsonValidator.java | 4 - .../cli/validate/DruidJsonValidatorTest.java | 1 - website/.spelling | 24 +- 155 files changed, 534 insertions(+), 3646 deletions(-) rename extensions-core/mysql-metadata-storage/src/main/java/org/apache/druid/{firehose/sql/MySQLFirehoseDatabaseConnector.java => metadata/input/MySQLInputSourceDatabaseConnector.java} (91%) rename extensions-core/mysql-metadata-storage/src/test/java/org/apache/druid/{firehose/sql/MySQLFirehoseDatabaseConnectorTest.java => metadata/input/MySQLInputSourceDatabaseConnectorTest.java} (92%) rename extensions-core/postgresql-metadata-storage/src/main/java/org/apache/druid/{firehose/PostgresqlFirehoseDatabaseConnector.java => metadata/input/PostgresqlInputSourceDatabaseConnector.java} (87%) rename extensions-core/postgresql-metadata-storage/src/test/java/org/apache/druid/{firehose/PostgresqlFirehoseDatabaseConnectorTest.java => metadata/input/PostgresqlInputSourceDatabaseConnectorTest.java} (90%) rename server/src/test/java/org/apache/druid/segment/realtime/firehose/IngestSegmentFirehoseTest.java => indexing-hadoop/src/test/java/org/apache/druid/indexer/hadoop/DatasourceRecordReaderSegmentReaderTest.java (92%) delete mode 100644 indexing-service/src/main/java/org/apache/druid/guice/IndexingServiceFirehoseModule.java rename indexing-service/src/main/java/org/apache/druid/indexing/{firehose => input}/WindowedSegmentId.java (98%) delete mode 100644 indexing-service/src/test/java/org/apache/druid/indexing/common/TestFirehose.java rename indexing-service/src/test/java/org/apache/druid/indexing/{firehose => input}/WindowedSegmentIdTest.java (96%) delete mode 100644 integration-tests/src/main/java/org/apache/druid/testing/clients/EventReceiverFirehoseTestClient.java delete mode 100644 processing/src/main/java/org/apache/druid/data/input/Firehose.java delete mode 100644 processing/src/main/java/org/apache/druid/data/input/FirehoseFactory.java delete mode 100644 server/src/main/java/org/apache/druid/guice/FirehoseModule.java rename server/src/main/java/org/apache/druid/metadata/{SQLFirehoseDatabaseConnector.java => SQLInputSourceDatabaseConnector.java} (98%) rename server/src/main/java/org/apache/druid/segment/realtime/{firehose => }/ChatHandler.java (95%) rename server/src/main/java/org/apache/druid/segment/realtime/{firehose => }/ChatHandlerProvider.java (97%) rename server/src/main/java/org/apache/druid/segment/realtime/{firehose => }/ChatHandlerResource.java (98%) rename server/src/main/java/org/apache/druid/segment/realtime/{firehose => }/ChatHandlers.java (97%) rename server/src/main/java/org/apache/druid/segment/realtime/{firehose => }/NoopChatHandlerProvider.java (96%) rename server/src/main/java/org/apache/druid/segment/realtime/{firehose => }/ServiceAnnouncingChatHandlerProvider.java (98%) rename server/src/main/java/org/apache/druid/segment/realtime/{firehose => }/WindowedStorageAdapter.java (96%) delete mode 100644 server/src/main/java/org/apache/druid/segment/realtime/firehose/ClippedFirehoseFactory.java delete mode 100644 server/src/main/java/org/apache/druid/segment/realtime/firehose/EventReceiver.java delete mode 100644 server/src/main/java/org/apache/druid/segment/realtime/firehose/EventReceiverFirehoseFactory.java delete mode 100644 server/src/main/java/org/apache/druid/segment/realtime/firehose/FixedCountFirehoseFactory.java delete mode 100644 server/src/main/java/org/apache/druid/segment/realtime/firehose/IngestSegmentFirehose.java delete mode 100644 server/src/main/java/org/apache/druid/segment/realtime/firehose/PredicateFirehose.java delete mode 100644 server/src/main/java/org/apache/druid/segment/realtime/firehose/TimedShutoffFirehoseFactory.java rename server/src/main/java/org/apache/druid/segment/realtime/{firehose => }/package-info.java (94%) delete mode 100644 server/src/main/java/org/apache/druid/server/metrics/EventReceiverFirehoseMetric.java delete mode 100644 server/src/main/java/org/apache/druid/server/metrics/EventReceiverFirehoseMonitor.java delete mode 100644 server/src/main/java/org/apache/druid/server/metrics/EventReceiverFirehoseRegister.java delete mode 100644 server/src/test/java/org/apache/druid/guice/FirehoseModuleTest.java rename server/src/test/java/org/apache/druid/segment/realtime/{firehose => }/ChatHandlerResourceTest.java (97%) rename server/src/test/java/org/apache/druid/segment/realtime/{firehose => }/ServiceAnnouncingChatHandlerProviderTest.java (99%) delete mode 100644 server/src/test/java/org/apache/druid/segment/realtime/firehose/EventReceiverFirehoseIdleTest.java delete mode 100644 server/src/test/java/org/apache/druid/segment/realtime/firehose/EventReceiverFirehoseTest.java diff --git a/docs/configuration/extensions.md b/docs/configuration/extensions.md index d396bc29000d..bc7a05a4ae23 100644 --- a/docs/configuration/extensions.md +++ b/docs/configuration/extensions.md @@ -80,7 +80,7 @@ All of these community extensions can be downloaded using [pull-deps](../operati |aliyun-oss-extensions|Aliyun OSS deep storage |[link](../development/extensions-contrib/aliyun-oss-extensions.md)| |ambari-metrics-emitter|Ambari Metrics Emitter |[link](../development/extensions-contrib/ambari-metrics-emitter.md)| |druid-cassandra-storage|Apache Cassandra deep storage.|[link](../development/extensions-contrib/cassandra.md)| -|druid-cloudfiles-extensions|Rackspace Cloudfiles deep storage and firehose.|[link](../development/extensions-contrib/cloudfiles.md)| +|druid-cloudfiles-extensions|Rackspace Cloudfiles deep storage.|[link](../development/extensions-contrib/cloudfiles.md)| |druid-compressed-bigdecimal|Compressed Big Decimal Type | [link](../development/extensions-contrib/compressed-big-decimal.md)| |druid-ddsketch|Support for DDSketch approximate quantiles based on [DDSketch](https://github.com/datadog/sketches-java) | [link](../development/extensions-contrib/ddsketch-quantiles.md)| |druid-deltalake-extensions|Support for ingesting Delta Lake tables.|[link](../development/extensions-contrib/delta-lake.md)| diff --git a/docs/configuration/index.md b/docs/configuration/index.md index c627e9fd7f07..39c14b77c461 100644 --- a/docs/configuration/index.md +++ b/docs/configuration/index.md @@ -395,7 +395,6 @@ Metric monitoring is an essential part of Druid operations. The following monito |`org.apache.druid.java.util.metrics.CgroupCpuSetMonitor`|Reports CPU core/HT and memory node allocations as per the `cpuset` cgroup.| |`org.apache.druid.java.util.metrics.CgroupDiskMonitor`|Reports disk statistic as per the blkio cgroup.| |`org.apache.druid.java.util.metrics.CgroupMemoryMonitor`|Reports memory statistic as per the memory cgroup.| -|`org.apache.druid.server.metrics.EventReceiverFirehoseMonitor`|Reports how many events have been queued in the EventReceiverFirehose.| |`org.apache.druid.server.metrics.HistoricalMetricsMonitor`|Reports statistics on Historical services. Available only on Historical services.| |`org.apache.druid.server.metrics.SegmentStatsMonitor` | **EXPERIMENTAL** Reports statistics about segments on Historical services. Available only on Historical services. Not to be used when lazy loading is configured.| |`org.apache.druid.server.metrics.QueryCountStatsMonitor`|Reports how many queries have been successful/failed/interrupted.| @@ -607,7 +606,7 @@ the [HDFS input source](../ingestion/input-sources.md#hdfs-input-source). |Property|Possible values|Description|Default| |--------|---------------|-----------|-------| -|`druid.ingestion.hdfs.allowedProtocols`|List of protocols|Allowed protocols for the HDFS input source and HDFS firehose.|`["hdfs"]`| +|`druid.ingestion.hdfs.allowedProtocols`|List of protocols|Allowed protocols for the HDFS input source.|`["hdfs"]`| #### HTTP input source @@ -616,7 +615,7 @@ the [HTTP input source](../ingestion/input-sources.md#http-input-source). |Property|Possible values|Description|Default| |--------|---------------|-----------|-------| -|`druid.ingestion.http.allowedProtocols`|List of protocols|Allowed protocols for the HTTP input source and HTTP firehose.|`["http", "https"]`| +|`druid.ingestion.http.allowedProtocols`|List of protocols|Allowed protocols for the HTTP input source.|`["http", "https"]`| ### External data access security configuration diff --git a/docs/development/extensions-contrib/cloudfiles.md b/docs/development/extensions-contrib/cloudfiles.md index 83a1d0c7e10b..d4e7592ee7f7 100644 --- a/docs/development/extensions-contrib/cloudfiles.md +++ b/docs/development/extensions-contrib/cloudfiles.md @@ -40,59 +40,3 @@ To use this Apache Druid extension, [include](../../configuration/extensions.md# |`druid.cloudfiles.apiKey`||Rackspace Cloud API key.|Must be set.| |`druid.cloudfiles.provider`|rackspace-cloudfiles-us,rackspace-cloudfiles-uk|Name of the provider depending on the region.|Must be set.| |`druid.cloudfiles.useServiceNet`|true,false|Whether to use the internal service net.|true| - -## Firehose - - - -#### StaticCloudFilesFirehose - -This firehose ingests events, similar to the StaticAzureBlobStoreFirehose, but from Rackspace's Cloud Files. - -Data is newline delimited, with one JSON object per line and parsed as per the `InputRowParser` configuration. - -The storage account is shared with the one used for Rackspace's Cloud Files deep storage functionality, but blobs can be in a different region and container. - -As with the Azure blobstore, it is assumed to be gzipped if the extension ends in .gz - -This firehose is _splittable_ and can be used by [native parallel index tasks](../../ingestion/native-batch.md). -Since each split represents an object in this firehose, each worker task of `index_parallel` will read an object. - -Sample spec: - -```json -"firehose" : { - "type" : "static-cloudfiles", - "blobs": [ - { - "region": "DFW" - "container": "container", - "path": "/path/to/your/file.json" - }, - { - "region": "ORD" - "container": "anothercontainer", - "path": "/another/path.json" - } - ] -} -``` -This firehose provides caching and prefetching features. In IndexTask, a firehose can be read twice if intervals or -shardSpecs are not specified, and, in this case, caching can be useful. Prefetching is preferred when direct scan of objects is slow. - -|property|description|default|required?| -|--------|-----------|-------|---------| -|type|This should be `static-cloudfiles`.|N/A|yes| -|blobs|JSON array of Cloud Files blobs.|N/A|yes| -|maxCacheCapacityBytes|Maximum size of the cache space in bytes. 0 means disabling cache.|1073741824|no| -|maxCacheCapacityBytes|Maximum size of the cache space in bytes. 0 means disabling cache. Cached files are not removed until the ingestion task completes.|1073741824|no| -|maxFetchCapacityBytes|Maximum size of the fetch space in bytes. 0 means disabling prefetch. Prefetched files are removed immediately once they are read.|1073741824|no| -|fetchTimeout|Timeout for fetching a Cloud Files object.|60000|no| -|maxFetchRetry|Maximum retry for fetching a Cloud Files object.|3|no| - -Cloud Files Blobs: - -|property|description|default|required?| -|--------|-----------|-------|---------| -|container|Name of the Cloud Files container|N/A|yes| -|path|The path where data is located.|N/A|yes| diff --git a/docs/development/extensions-core/postgresql.md b/docs/development/extensions-core/postgresql.md index e0d9337b2ccb..919bf372b844 100644 --- a/docs/development/extensions-core/postgresql.md +++ b/docs/development/extensions-core/postgresql.md @@ -87,7 +87,7 @@ In most cases, the configuration options map directly to the [postgres JDBC conn | `druid.metadata.postgres.ssl.sslPasswordCallback` | The classname of the SSL password provider. | none | no | | `druid.metadata.postgres.dbTableSchema` | druid meta table schema | `public` | no | -### PostgreSQL Firehose +### PostgreSQL InputSource The PostgreSQL extension provides an implementation of an [SQL input source](../../ingestion/input-sources.md) which can be used to ingest data into Druid from a PostgreSQL database. diff --git a/docs/development/overview.md b/docs/development/overview.md index 5ff77af07cf4..11c67ddfa2c4 100644 --- a/docs/development/overview.md +++ b/docs/development/overview.md @@ -53,8 +53,17 @@ Most of the coordination logic for (real-time) ingestion is in the Druid indexin ## Real-time Ingestion -Druid loads data through `FirehoseFactory.java` classes. Firehoses often wrap other firehoses, where, similar to the design of the -query runners, each firehose adds a layer of logic, and the persist and hand-off logic is in `RealtimePlumber.java`. +Druid streaming tasks are based on the 'seekable stream' classes such as `SeekableStreamSupervisor.java`, +`SeekableStreamIndexTask.java`, and `SeekableStreamIndexTaskRunner.java`. The data processing happens through +`StreamAppenderator.java`, and the persist and hand-off logic is in `StreamAppenderatorDriver.java`. + +## Native Batch Ingestion + +Druid native batch ingestion main task types are based on `AbstractBatchTask.java` and `AbstractBatchSubtask.java`. +Parallel processing uses `ParallelIndexSupervisorTask.java`, which spawns subtasks to perform various operations such +as data analysis and partitioning depending on the task specification. Segment generation happens in +`SinglePhaseSubTask.java`, `PartialHashSegmentGenerateTask.java`, or `PartialRangeSegmentGenerateTask.java` through +`BatchAppenderator`, and the persist and hand-off logic is in `BatchAppenderatorDriver.java`. ## Hadoop-based Batch Ingestion diff --git a/docs/ingestion/native-batch-firehose.md b/docs/ingestion/native-batch-firehose.md index db2b3e8779e6..16e9634ff297 100644 --- a/docs/ingestion/native-batch-firehose.md +++ b/docs/ingestion/native-batch-firehose.md @@ -24,319 +24,5 @@ sidebar_label: "Firehose (deprecated)" --> :::info - Firehose ingestion is deprecated. See [Migrate from firehose to input source ingestion](../operations/migrate-from-firehose-ingestion.md) for instructions on migrating from firehose ingestion to using native batch ingestion input sources. + Firehose ingestion has been removed in Druid 26.0. See [Migrate from firehose to input source ingestion](../operations/migrate-from-firehose-ingestion.md) for instructions on migrating from firehose ingestion to using native batch ingestion input sources. ::: - -There are several firehoses readily available in Druid, some are meant for examples, others can be used directly in a production environment. - -## StaticS3Firehose - -You need to include the [`druid-s3-extensions`](../development/extensions-core/s3.md) as an extension to use the StaticS3Firehose. - -This firehose ingests events from a predefined list of S3 objects. -This firehose is _splittable_ and can be used by the [Parallel task](./native-batch.md). -Since each split represents an object in this firehose, each worker task of `index_parallel` will read an object. - -Sample spec: - -```json -"firehose" : { - "type" : "static-s3", - "uris": ["s3://foo/bar/file.gz", "s3://bar/foo/file2.gz"] -} -``` - -This firehose provides caching and prefetching features. In the Simple task, a firehose can be read twice if intervals or -shardSpecs are not specified, and, in this case, caching can be useful. Prefetching is preferred when direct scan of objects is slow. -Note that prefetching or caching isn't that useful in the Parallel task. - -|property|description|default|required?| -|--------|-----------|-------|---------| -|type|This should be `static-s3`.|None|yes| -|uris|JSON array of URIs where s3 files to be ingested are located.|None|`uris` or `prefixes` must be set| -|prefixes|JSON array of URI prefixes for the locations of s3 files to be ingested.|None|`uris` or `prefixes` must be set| -|maxCacheCapacityBytes|Maximum size of the cache space in bytes. 0 means disabling cache. Cached files are not removed until the ingestion task completes.|1073741824|no| -|maxFetchCapacityBytes|Maximum size of the fetch space in bytes. 0 means disabling prefetch. Prefetched files are removed immediately once they are read.|1073741824|no| -|prefetchTriggerBytes|Threshold to trigger prefetching s3 objects.|maxFetchCapacityBytes / 2|no| -|fetchTimeout|Timeout for fetching an s3 object.|60000|no| -|maxFetchRetry|Maximum retry for fetching an s3 object.|3|no| - -## StaticGoogleBlobStoreFirehose - -You need to include the [`druid-google-extensions`](../development/extensions-core/google.md) as an extension to use the StaticGoogleBlobStoreFirehose. - -This firehose ingests events, similar to the StaticS3Firehose, but from an Google Cloud Store. - -As with the S3 blobstore, it is assumed to be gzipped if the extension ends in .gz - -This firehose is _splittable_ and can be used by the [Parallel task](./native-batch.md). -Since each split represents an object in this firehose, each worker task of `index_parallel` will read an object. - -Sample spec: - -```json -"firehose" : { - "type" : "static-google-blobstore", - "blobs": [ - { - "bucket": "foo", - "path": "/path/to/your/file.json" - }, - { - "bucket": "bar", - "path": "/another/path.json" - } - ] -} -``` - -This firehose provides caching and prefetching features. In the Simple task, a firehose can be read twice if intervals or -shardSpecs are not specified, and, in this case, caching can be useful. Prefetching is preferred when direct scan of objects is slow. -Note that prefetching or caching isn't that useful in the Parallel task. - -|property|description|default|required?| -|--------|-----------|-------|---------| -|type|This should be `static-google-blobstore`.|None|yes| -|blobs|JSON array of Google Blobs.|None|yes| -|maxCacheCapacityBytes|Maximum size of the cache space in bytes. 0 means disabling cache. Cached files are not removed until the ingestion task completes.|1073741824|no| -|maxFetchCapacityBytes|Maximum size of the fetch space in bytes. 0 means disabling prefetch. Prefetched files are removed immediately once they are read.|1073741824|no| -|prefetchTriggerBytes|Threshold to trigger prefetching Google Blobs.|maxFetchCapacityBytes / 2|no| -|fetchTimeout|Timeout for fetching a Google Blob.|60000|no| -|maxFetchRetry|Maximum retry for fetching a Google Blob.|3|no| - -Google Blobs: - -|property|description|default|required?| -|--------|-----------|-------|---------| -|bucket|Name of the Google Cloud bucket|None|yes| -|path|The path where data is located.|None|yes| - -## HDFSFirehose - -You need to include the [`druid-hdfs-storage`](../development/extensions-core/hdfs.md) as an extension to use the HDFSFirehose. - -This firehose ingests events from a predefined list of files from the HDFS storage. -This firehose is _splittable_ and can be used by the [Parallel task](./native-batch.md). -Since each split represents an HDFS file, each worker task of `index_parallel` will read files. - -Sample spec: - -```json -"firehose" : { - "type" : "hdfs", - "paths": "/foo/bar,/foo/baz" -} -``` - -This firehose provides caching and prefetching features. During native batch indexing, a firehose can be read twice if -`intervals` are not specified, and, in this case, caching can be useful. Prefetching is preferred when direct scanning -of files is slow. -Note that prefetching or caching isn't that useful in the Parallel task. - -|Property|Description|Default| -|--------|-----------|-------| -|type|This should be `hdfs`.|none (required)| -|paths|HDFS paths. Can be either a JSON array or comma-separated string of paths. Wildcards like `*` are supported in these paths.|none (required)| -|maxCacheCapacityBytes|Maximum size of the cache space in bytes. 0 means disabling cache. Cached files are not removed until the ingestion task completes.|1073741824| -|maxFetchCapacityBytes|Maximum size of the fetch space in bytes. 0 means disabling prefetch. Prefetched files are removed immediately once they are read.|1073741824| -|prefetchTriggerBytes|Threshold to trigger prefetching files.|maxFetchCapacityBytes / 2| -|fetchTimeout|Timeout for fetching each file.|60000| -|maxFetchRetry|Maximum number of retries for fetching each file.|3| - -You can also ingest from other storage using the HDFS firehose if the HDFS client supports that storage. -However, if you want to ingest from cloud storage, consider using the service-specific input source for your data storage. -If you want to use a non-hdfs protocol with the HDFS firehose, you need to include the protocol you want -in `druid.ingestion.hdfs.allowedProtocols`. See [HDFS firehose security configuration](../configuration/index.md#hdfs-input-source) for more details. - -## LocalFirehose - -This Firehose can be used to read the data from files on local disk, and is mainly intended for proof-of-concept testing, and works with `string` typed parsers. -This Firehose is _splittable_ and can be used by [native parallel index tasks](native-batch.md). -Since each split represents a file in this Firehose, each worker task of `index_parallel` will read a file. -A sample local Firehose spec is shown below: - -```json -{ - "type": "local", - "filter" : "*.csv", - "baseDir": "/data/directory" -} -``` - -|property|description|required?| -|--------|-----------|---------| -|type|This should be "local".|yes| -|filter|A wildcard filter for files. See [here](http://commons.apache.org/proper/commons-io/apidocs/org/apache/commons/io/filefilter/WildcardFileFilter) for more information.|yes| -|baseDir|directory to search recursively for files to be ingested. |yes| - - - -## HttpFirehose - -This Firehose can be used to read the data from remote sites via HTTP, and works with `string` typed parsers. -This Firehose is _splittable_ and can be used by [native parallel index tasks](native-batch.md). -Since each split represents a file in this Firehose, each worker task of `index_parallel` will read a file. -A sample HTTP Firehose spec is shown below: - -```json -{ - "type": "http", - "uris": ["http://example.com/uri1", "http://example2.com/uri2"] -} -``` - -You can only use protocols listed in the `druid.ingestion.http.allowedProtocols` property as HTTP firehose input sources. -The `http` and `https` protocols are allowed by default. See [HTTP firehose security configuration](../configuration/index.md#http-input-source) for more details. - -The below configurations can be optionally used if the URIs specified in the spec require a Basic Authentication Header. -Omitting these fields from your spec will result in HTTP requests with no Basic Authentication Header. - -|property|description|default| -|--------|-----------|-------| -|httpAuthenticationUsername|Username to use for authentication with specified URIs|None| -|httpAuthenticationPassword|PasswordProvider to use with specified URIs|None| - -Example with authentication fields using the DefaultPassword provider (this requires the password to be in the ingestion spec): - -```json -{ - "type": "http", - "uris": ["http://example.com/uri1", "http://example2.com/uri2"], - "httpAuthenticationUsername": "username", - "httpAuthenticationPassword": "password123" -} -``` - -You can also use the other existing Druid PasswordProviders. Here is an example using the EnvironmentVariablePasswordProvider: - -```json -{ - "type": "http", - "uris": ["http://example.com/uri1", "http://example2.com/uri2"], - "httpAuthenticationUsername": "username", - "httpAuthenticationPassword": { - "type": "environment", - "variable": "HTTP_FIREHOSE_PW" - } -} -``` - -The below configurations can optionally be used for tuning the Firehose performance. -Note that prefetching or caching isn't that useful in the Parallel task. - -|property|description|default| -|--------|-----------|-------| -|maxCacheCapacityBytes|Maximum size of the cache space in bytes. 0 means disabling cache. Cached files are not removed until the ingestion task completes.|1073741824| -|maxFetchCapacityBytes|Maximum size of the fetch space in bytes. 0 means disabling prefetch. Prefetched files are removed immediately once they are read.|1073741824| -|prefetchTriggerBytes|Threshold to trigger prefetching HTTP objects.|maxFetchCapacityBytes / 2| -|fetchTimeout|Timeout for fetching an HTTP object.|60000| -|maxFetchRetry|Maximum retries for fetching an HTTP object.|3| - - - -## IngestSegmentFirehose - -This Firehose can be used to read the data from existing druid segments, potentially using a new schema and changing the name, dimensions, metrics, rollup, etc. of the segment. -This Firehose is _splittable_ and can be used by [native parallel index tasks](native-batch.md). -This firehose will accept any type of parser, but will only utilize the list of dimensions and the timestamp specification. - A sample ingest Firehose spec is shown below: - -```json -{ - "type": "ingestSegment", - "dataSource": "wikipedia", - "interval": "2013-01-01/2013-01-02" -} -``` - -|property|description|required?| -|--------|-----------|---------| -|type|This should be "ingestSegment".|yes| -|dataSource|A String defining the data source to fetch rows from, very similar to a table in a relational database|yes| -|interval|A String representing the ISO-8601 interval. This defines the time range to fetch the data over.|yes| -|dimensions|The list of dimensions to select. If left empty, no dimensions are returned. If left null or not defined, all dimensions are returned. |no| -|metrics|The list of metrics to select. If left empty, no metrics are returned. If left null or not defined, all metrics are selected.|no| -|filter| See [Filters](../querying/filters.md)|no| -|maxInputSegmentBytesPerTask|Deprecated. Use [Segments Split Hint Spec](./native-batch.md#segments-split-hint-spec) instead. When used with the native parallel index task, the maximum number of bytes of input segments to process in a single task. If a single segment is larger than this number, it will be processed by itself in a single task (input segments are never split across tasks). Defaults to 150MB.|no| - - - -## SqlFirehose - -This Firehose can be used to ingest events residing in an RDBMS. The database connection information is provided as part of the ingestion spec. -For each query, the results are fetched locally and indexed. -If there are multiple queries from which data needs to be indexed, queries are prefetched in the background, up to `maxFetchCapacityBytes` bytes. -This Firehose is _splittable_ and can be used by [native parallel index tasks](native-batch.md). -This firehose will accept any type of parser, but will only utilize the list of dimensions and the timestamp specification. See the extension documentation for more detailed ingestion examples. - -Requires one of the following extensions: - * [MySQL Metadata Store](../development/extensions-core/mysql.md). - * [PostgreSQL Metadata Store](../development/extensions-core/postgresql.md). - - -```json -{ - "type": "sql", - "database": { - "type": "mysql", - "connectorConfig": { - "connectURI": "jdbc:mysql://host:port/schema", - "user": "user", - "password": "password" - } - }, - "sqls": ["SELECT * FROM table1", "SELECT * FROM table2"] -} -``` - -|property|description|default|required?| -|--------|-----------|-------|---------| -|type|This should be "sql".||Yes| -|database|Specifies the database connection details. The database type corresponds to the extension that supplies the `connectorConfig` support. The specified extension must be loaded into Druid:



You can selectively allow JDBC properties in `connectURI`. See [JDBC connections security config](../configuration/index.md#jdbc-connections-to-external-databases) for more details.||Yes| -|maxCacheCapacityBytes|Maximum size of the cache space in bytes. 0 means disabling cache. Cached files are not removed until the ingestion task completes.|1073741824|No| -|maxFetchCapacityBytes|Maximum size of the fetch space in bytes. 0 means disabling prefetch. Prefetched files are removed immediately once they are read.|1073741824|No| -|prefetchTriggerBytes|Threshold to trigger prefetching SQL result objects.|maxFetchCapacityBytes / 2|No| -|fetchTimeout|Timeout for fetching the result set.|60000|No| -|foldCase|Toggle case folding of database column names. This may be enabled in cases where the database returns case insensitive column names in query results.|false|No| -|sqls|List of SQL queries where each SQL query would retrieve the data to be indexed.||Yes| - -### Database - -|property|description|default|required?| -|--------|-----------|-------|---------| -|type|The type of database to query. Valid values are `mysql` and `postgresql`_||Yes| -|connectorConfig|Specify the database connection properties via `connectURI`, `user` and `password`||Yes| - -## InlineFirehose - -This Firehose can be used to read the data inlined in its own spec. -It can be used for demos or for quickly testing out parsing and schema, and works with `string` typed parsers. -A sample inline Firehose spec is shown below: - -```json -{ - "type": "inline", - "data": "0,values,formatted\n1,as,CSV" -} -``` - -|property|description|required?| -|--------|-----------|---------| -|type|This should be "inline".|yes| -|data|Inlined data to ingest.|yes| - -## CombiningFirehose - -This Firehose can be used to combine and merge data from a list of different Firehoses. - -```json -{ - "type": "combining", - "delegates": [ { firehose1 }, { firehose2 }, ... ] -} -``` - -|property|description|required?| -|--------|-----------|---------| -|type|This should be "combining"|yes| -|delegates|List of Firehoses to combine data from|yes| \ No newline at end of file diff --git a/docs/operations/metrics.md b/docs/operations/metrics.md index 1d37169684e1..b5be94ba4ba7 100644 --- a/docs/operations/metrics.md +++ b/docs/operations/metrics.md @@ -428,15 +428,6 @@ These metrics are available only when `druid.zk.service.enabled = true`. |`zk/connected`|Indicator of connection status. `1` for connected, `0` for disconnected. Emitted once per monitor period.|None|1| |`zk/reconnect/time`|Amount of time, in milliseconds, that a server was disconnected from ZooKeeper before reconnecting. Emitted on reconnection. Not emitted if connection to ZooKeeper is permanently lost, because in this case, there is no reconnection.|None|Not present| -### EventReceiverFirehose - -The following metric is only available if the `EventReceiverFirehoseMonitor` module is included. - -|Metric|Description|Dimensions|Normal value| -|------|-----------|----------|------------| -|`ingest/events/buffered`|Number of events queued in the `EventReceiverFirehose` buffer.|`serviceName`, `dataSource`, `taskId`, `taskType`, `bufferCapacity`|Equal to the current number of events in the buffer queue.| -|`ingest/bytes/received`|Number of bytes received by the `EventReceiverFirehose`.|`serviceName`, `dataSource`, `taskId`, `taskType`|Varies| - ## Sys [Deprecated] > SysMonitor is now deprecated and will be removed in future releases. diff --git a/docs/operations/migrate-from-firehose-ingestion.md b/docs/operations/migrate-from-firehose-ingestion.md index f470324b7f4e..540685a717f1 100644 --- a/docs/operations/migrate-from-firehose-ingestion.md +++ b/docs/operations/migrate-from-firehose-ingestion.md @@ -23,9 +23,7 @@ sidebar_label: "Migrate from firehose" ~ under the License. --> -Apache deprecated support for Druid firehoses in version 0.17. Support for firehose ingestion will be removed in version 26.0. - -If you're using a firehose for batch ingestion, we strongly recommend that you follow the instructions on this page to transition to using native batch ingestion input sources as soon as possible. +Apache deprecated support for Druid firehoses in version 0.17. Support for firehose ingestion was removed in version 26.0. Firehose ingestion doesn't work with newer Druid versions, so you must be using an ingestion spec with a defined input source before you upgrade. diff --git a/docs/operations/security-overview.md b/docs/operations/security-overview.md index 279a1327b97d..96389c8a72c7 100644 --- a/docs/operations/security-overview.md +++ b/docs/operations/security-overview.md @@ -52,7 +52,7 @@ The following recommendations apply to the network where Druid runs: The following recommendation applies to Druid's authorization and authentication model: * Only grant `WRITE` permissions to any `DATASOURCE` to trusted users. Druid's trust model assumes those users have the same privileges as the operating system user that runs the web console process. Additionally, users with `WRITE` permissions can make changes to datasources and they have access to both task and supervisor update (POST) APIs which may affect ingestion. * Only grant `STATE READ`, `STATE WRITE`, `CONFIG WRITE`, and `DATASOURCE WRITE` permissions to highly-trusted users. These permissions allow users to access resources on behalf of the Druid server process regardless of the datasource. -* If your Druid client application allows less-trusted users to control the input source or firehose of an ingestion task, validate the URLs from the users. It is possible to point unchecked URLs to other locations and resources within your network or local file system. +* If your Druid client application allows less-trusted users to control the input source of an ingestion task, validate the URLs from the users. It is possible to point unchecked URLs to other locations and resources within your network or local file system. ## Enable TLS diff --git a/extensions-contrib/kubernetes-overlord-extensions/src/test/java/org/apache/druid/k8s/overlord/common/K8sTestUtils.java b/extensions-contrib/kubernetes-overlord-extensions/src/test/java/org/apache/druid/k8s/overlord/common/K8sTestUtils.java index 04b332aac850..b3fda99b222e 100644 --- a/extensions-contrib/kubernetes-overlord-extensions/src/test/java/org/apache/druid/k8s/overlord/common/K8sTestUtils.java +++ b/extensions-contrib/kubernetes-overlord-extensions/src/test/java/org/apache/druid/k8s/overlord/common/K8sTestUtils.java @@ -79,7 +79,6 @@ public static Task getTask() null ), new IndexTask.IndexIOConfig( - null, new LocalInputSource(new File("lol"), "rofl"), new NoopInputFormat(), true, diff --git a/extensions-contrib/kubernetes-overlord-extensions/src/test/java/org/apache/druid/k8s/overlord/taskadapter/DruidPeonClientIntegrationTest.java b/extensions-contrib/kubernetes-overlord-extensions/src/test/java/org/apache/druid/k8s/overlord/taskadapter/DruidPeonClientIntegrationTest.java index 241b4d9fc68f..e2d97accc97f 100644 --- a/extensions-contrib/kubernetes-overlord-extensions/src/test/java/org/apache/druid/k8s/overlord/taskadapter/DruidPeonClientIntegrationTest.java +++ b/extensions-contrib/kubernetes-overlord-extensions/src/test/java/org/apache/druid/k8s/overlord/taskadapter/DruidPeonClientIntegrationTest.java @@ -19,7 +19,6 @@ package org.apache.druid.k8s.overlord.taskadapter; -import com.fasterxml.jackson.databind.Module; import com.fasterxml.jackson.databind.ObjectMapper; import com.fasterxml.jackson.databind.jsontype.NamedType; import io.fabric8.kubernetes.api.model.Pod; @@ -27,7 +26,6 @@ import io.fabric8.kubernetes.api.model.batch.v1.Job; import org.apache.commons.io.FileUtils; import org.apache.commons.io.IOUtils; -import org.apache.druid.guice.FirehoseModule; import org.apache.druid.indexing.common.TestUtils; import org.apache.druid.indexing.common.config.TaskConfig; import org.apache.druid.indexing.common.config.TaskConfigBuilder; @@ -83,9 +81,6 @@ public void setup() { TestUtils utils = new TestUtils(); jsonMapper = utils.getTestObjectMapper(); - for (Module jacksonModule : new FirehoseModule().getJacksonModules()) { - jsonMapper.registerModule(jacksonModule); - } jsonMapper.registerSubtypes( new NamedType(ParallelIndexTuningConfig.class, "index_parallel"), new NamedType(IndexTask.IndexTuningConfig.class, "index") diff --git a/extensions-contrib/kubernetes-overlord-extensions/src/test/java/org/apache/druid/k8s/overlord/taskadapter/K8sTaskAdapterTest.java b/extensions-contrib/kubernetes-overlord-extensions/src/test/java/org/apache/druid/k8s/overlord/taskadapter/K8sTaskAdapterTest.java index 102565efc35a..f8e7186a0263 100644 --- a/extensions-contrib/kubernetes-overlord-extensions/src/test/java/org/apache/druid/k8s/overlord/taskadapter/K8sTaskAdapterTest.java +++ b/extensions-contrib/kubernetes-overlord-extensions/src/test/java/org/apache/druid/k8s/overlord/taskadapter/K8sTaskAdapterTest.java @@ -19,7 +19,6 @@ package org.apache.druid.k8s.overlord.taskadapter; -import com.fasterxml.jackson.databind.Module; import com.fasterxml.jackson.databind.ObjectMapper; import com.fasterxml.jackson.databind.jsontype.NamedType; import com.google.common.base.Joiner; @@ -44,7 +43,6 @@ import org.apache.commons.lang.RandomStringUtils; import org.apache.commons.lang.StringUtils; import org.apache.druid.error.DruidException; -import org.apache.druid.guice.FirehoseModule; import org.apache.druid.indexing.common.TestUtils; import org.apache.druid.indexing.common.config.TaskConfig; import org.apache.druid.indexing.common.config.TaskConfigBuilder; @@ -101,9 +99,6 @@ public K8sTaskAdapterTest() { TestUtils utils = new TestUtils(); jsonMapper = utils.getTestObjectMapper(); - for (Module jacksonModule : new FirehoseModule().getJacksonModules()) { - jsonMapper.registerModule(jacksonModule); - } jsonMapper.registerSubtypes( new NamedType(ParallelIndexTuningConfig.class, "index_parallel"), new NamedType(IndexTask.IndexTuningConfig.class, "index") diff --git a/extensions-contrib/kubernetes-overlord-extensions/src/test/java/org/apache/druid/k8s/overlord/taskadapter/MultiContainerTaskAdapterTest.java b/extensions-contrib/kubernetes-overlord-extensions/src/test/java/org/apache/druid/k8s/overlord/taskadapter/MultiContainerTaskAdapterTest.java index 58993b9a6a00..9308835d967f 100644 --- a/extensions-contrib/kubernetes-overlord-extensions/src/test/java/org/apache/druid/k8s/overlord/taskadapter/MultiContainerTaskAdapterTest.java +++ b/extensions-contrib/kubernetes-overlord-extensions/src/test/java/org/apache/druid/k8s/overlord/taskadapter/MultiContainerTaskAdapterTest.java @@ -19,7 +19,6 @@ package org.apache.druid.k8s.overlord.taskadapter; -import com.fasterxml.jackson.databind.Module; import com.fasterxml.jackson.databind.ObjectMapper; import com.fasterxml.jackson.databind.jsontype.NamedType; import com.google.common.collect.ImmutableList; @@ -28,7 +27,6 @@ import io.fabric8.kubernetes.api.model.batch.v1.Job; import io.fabric8.kubernetes.client.KubernetesClient; import io.fabric8.kubernetes.client.server.mock.EnableKubernetesMockClient; -import org.apache.druid.guice.FirehoseModule; import org.apache.druid.indexing.common.TestUtils; import org.apache.druid.indexing.common.config.TaskConfig; import org.apache.druid.indexing.common.config.TaskConfigBuilder; @@ -67,9 +65,6 @@ public void setup() { TestUtils utils = new TestUtils(); jsonMapper = utils.getTestObjectMapper(); - for (Module jacksonModule : new FirehoseModule().getJacksonModules()) { - jsonMapper.registerModule(jacksonModule); - } jsonMapper.registerSubtypes( new NamedType(ParallelIndexTuningConfig.class, "index_parallel"), new NamedType(IndexTask.IndexTuningConfig.class, "index") diff --git a/extensions-contrib/kubernetes-overlord-extensions/src/test/java/org/apache/druid/k8s/overlord/taskadapter/SingleContainerTaskAdapterTest.java b/extensions-contrib/kubernetes-overlord-extensions/src/test/java/org/apache/druid/k8s/overlord/taskadapter/SingleContainerTaskAdapterTest.java index afc5299927fa..4cf9c3e4cbbc 100644 --- a/extensions-contrib/kubernetes-overlord-extensions/src/test/java/org/apache/druid/k8s/overlord/taskadapter/SingleContainerTaskAdapterTest.java +++ b/extensions-contrib/kubernetes-overlord-extensions/src/test/java/org/apache/druid/k8s/overlord/taskadapter/SingleContainerTaskAdapterTest.java @@ -19,14 +19,12 @@ package org.apache.druid.k8s.overlord.taskadapter; -import com.fasterxml.jackson.databind.Module; import com.fasterxml.jackson.databind.ObjectMapper; import com.fasterxml.jackson.databind.jsontype.NamedType; import io.fabric8.kubernetes.api.model.Pod; import io.fabric8.kubernetes.api.model.batch.v1.Job; import io.fabric8.kubernetes.client.KubernetesClient; import io.fabric8.kubernetes.client.server.mock.EnableKubernetesMockClient; -import org.apache.druid.guice.FirehoseModule; import org.apache.druid.indexing.common.TestUtils; import org.apache.druid.indexing.common.config.TaskConfig; import org.apache.druid.indexing.common.config.TaskConfigBuilder; @@ -66,9 +64,6 @@ public void setup() { TestUtils utils = new TestUtils(); jsonMapper = utils.getTestObjectMapper(); - for (Module jacksonModule : new FirehoseModule().getJacksonModules()) { - jsonMapper.registerModule(jacksonModule); - } jsonMapper.registerSubtypes( new NamedType(ParallelIndexTuningConfig.class, "index_parallel"), new NamedType(IndexTask.IndexTuningConfig.class, "index") diff --git a/extensions-contrib/materialized-view-maintenance/src/main/java/org/apache/druid/indexing/materializedview/MaterializedViewSupervisorSpec.java b/extensions-contrib/materialized-view-maintenance/src/main/java/org/apache/druid/indexing/materializedview/MaterializedViewSupervisorSpec.java index 7d59cb63ea5b..01039375259e 100644 --- a/extensions-contrib/materialized-view-maintenance/src/main/java/org/apache/druid/indexing/materializedview/MaterializedViewSupervisorSpec.java +++ b/extensions-contrib/materialized-view-maintenance/src/main/java/org/apache/druid/indexing/materializedview/MaterializedViewSupervisorSpec.java @@ -47,7 +47,7 @@ import org.apache.druid.query.aggregation.AggregatorFactory; import org.apache.druid.segment.indexing.DataSchema; import org.apache.druid.segment.indexing.granularity.ArbitraryGranularitySpec; -import org.apache.druid.segment.realtime.firehose.ChatHandlerProvider; +import org.apache.druid.segment.realtime.ChatHandlerProvider; import org.apache.druid.segment.transform.TransformSpec; import org.apache.druid.server.security.AuthorizerMapper; import org.apache.druid.timeline.DataSegment; diff --git a/extensions-contrib/materialized-view-maintenance/src/test/java/org/apache/druid/indexing/materializedview/MaterializedViewSupervisorSpecTest.java b/extensions-contrib/materialized-view-maintenance/src/test/java/org/apache/druid/indexing/materializedview/MaterializedViewSupervisorSpecTest.java index 14bd59871253..65e71f626e8e 100644 --- a/extensions-contrib/materialized-view-maintenance/src/test/java/org/apache/druid/indexing/materializedview/MaterializedViewSupervisorSpecTest.java +++ b/extensions-contrib/materialized-view-maintenance/src/test/java/org/apache/druid/indexing/materializedview/MaterializedViewSupervisorSpecTest.java @@ -40,8 +40,8 @@ import org.apache.druid.query.aggregation.LongSumAggregatorFactory; import org.apache.druid.query.expression.LookupEnabledTestExprMacroTable; import org.apache.druid.segment.TestHelper; -import org.apache.druid.segment.realtime.firehose.ChatHandlerProvider; -import org.apache.druid.segment.realtime.firehose.NoopChatHandlerProvider; +import org.apache.druid.segment.realtime.ChatHandlerProvider; +import org.apache.druid.segment.realtime.NoopChatHandlerProvider; import org.apache.druid.server.security.AuthorizerMapper; import org.easymock.EasyMock; import org.hamcrest.CoreMatchers; diff --git a/extensions-contrib/materialized-view-maintenance/src/test/java/org/apache/druid/indexing/materializedview/MaterializedViewSupervisorTest.java b/extensions-contrib/materialized-view-maintenance/src/test/java/org/apache/druid/indexing/materializedview/MaterializedViewSupervisorTest.java index 38f15a840dc2..ff1759195869 100644 --- a/extensions-contrib/materialized-view-maintenance/src/test/java/org/apache/druid/indexing/materializedview/MaterializedViewSupervisorTest.java +++ b/extensions-contrib/materialized-view-maintenance/src/test/java/org/apache/druid/indexing/materializedview/MaterializedViewSupervisorTest.java @@ -52,7 +52,7 @@ import org.apache.druid.segment.indexing.DataSchema; import org.apache.druid.segment.metadata.CentralizedDatasourceSchemaConfig; import org.apache.druid.segment.metadata.SegmentSchemaManager; -import org.apache.druid.segment.realtime.firehose.ChatHandlerProvider; +import org.apache.druid.segment.realtime.ChatHandlerProvider; import org.apache.druid.segment.transform.TransformSpec; import org.apache.druid.server.security.AuthorizerMapper; import org.apache.druid.timeline.DataSegment; diff --git a/extensions-contrib/prometheus-emitter/src/main/resources/defaultMetrics.json b/extensions-contrib/prometheus-emitter/src/main/resources/defaultMetrics.json index 7e625c9edb3d..9cb01646f209 100644 --- a/extensions-contrib/prometheus-emitter/src/main/resources/defaultMetrics.json +++ b/extensions-contrib/prometheus-emitter/src/main/resources/defaultMetrics.json @@ -201,9 +201,6 @@ "zk/connected" : { "dimensions" : [], "type" : "gauge", "help": "Indicator of connection status to zookeeper."}, "zk/reconnect/time" : { "dimensions" : [], "type" : "timer", "conversionFactor": 1000.0, "help": "Amount of time, in seconds, that a server was disconnected from ZooKeeper before reconnecting." }, - "ingest/events/buffered" : { "dimensions" : ["serviceName", "dataSource", "bufferCapacity"], "type" : "gauge", "help": "Number of events queued in the EventReceiverFirehose's buffer"}, - "ingest/bytes/received" : { "dimensions" : ["serviceName", "dataSource"], "type" : "gauge", "help": "Number of bytes received by the EventReceiverFirehose."}, - "sys/swap/free" : { "dimensions" : [], "type" : "gauge", "help": "Free swap"}, "sys/swap/max" : { "dimensions" : [], "type" : "gauge", "help": "Max swap"}, "sys/swap/pageIn" : { "dimensions" : [], "type" : "gauge", "help": "Paged in swap"}, diff --git a/extensions-core/azure-extensions/pom.xml b/extensions-core/azure-extensions/pom.xml index 88a1198b61e1..7f0f767af851 100644 --- a/extensions-core/azure-extensions/pom.xml +++ b/extensions-core/azure-extensions/pom.xml @@ -204,10 +204,6 @@ org.jacoco jacoco-maven-plugin - - - org/apache/druid/firehose/azure/**/* - BUNDLE diff --git a/extensions-core/kinesis-indexing-service/src/test/java/org/apache/druid/indexing/kinesis/KinesisIndexTaskSerdeTest.java b/extensions-core/kinesis-indexing-service/src/test/java/org/apache/druid/indexing/kinesis/KinesisIndexTaskSerdeTest.java index da04e3ab0a6d..60d8f686a28c 100644 --- a/extensions-core/kinesis-indexing-service/src/test/java/org/apache/druid/indexing/kinesis/KinesisIndexTaskSerdeTest.java +++ b/extensions-core/kinesis-indexing-service/src/test/java/org/apache/druid/indexing/kinesis/KinesisIndexTaskSerdeTest.java @@ -33,9 +33,9 @@ import org.apache.druid.initialization.Initialization; import org.apache.druid.segment.incremental.RowIngestionMetersFactory; import org.apache.druid.segment.indexing.DataSchema; +import org.apache.druid.segment.realtime.ChatHandlerProvider; +import org.apache.druid.segment.realtime.NoopChatHandlerProvider; import org.apache.druid.segment.realtime.appenderator.AppenderatorsManager; -import org.apache.druid.segment.realtime.firehose.ChatHandlerProvider; -import org.apache.druid.segment.realtime.firehose.NoopChatHandlerProvider; import org.apache.druid.server.security.Action; import org.apache.druid.server.security.Resource; import org.apache.druid.server.security.ResourceAction; diff --git a/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/indexing/IndexerControllerContext.java b/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/indexing/IndexerControllerContext.java index e960a8aea29c..0fef9d32e6d4 100644 --- a/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/indexing/IndexerControllerContext.java +++ b/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/indexing/IndexerControllerContext.java @@ -53,7 +53,7 @@ import org.apache.druid.query.QueryContext; import org.apache.druid.rpc.ServiceClientFactory; import org.apache.druid.rpc.indexing.OverlordClient; -import org.apache.druid.segment.realtime.firehose.ChatHandler; +import org.apache.druid.segment.realtime.ChatHandler; import org.apache.druid.server.DruidNode; import org.apache.druid.server.lookup.cache.LookupLoadingSpec; diff --git a/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/indexing/client/ControllerChatHandler.java b/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/indexing/client/ControllerChatHandler.java index bf3dd4a6bf14..d161b01bd0b6 100644 --- a/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/indexing/client/ControllerChatHandler.java +++ b/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/indexing/client/ControllerChatHandler.java @@ -22,7 +22,7 @@ import org.apache.druid.msq.exec.Controller; import org.apache.druid.msq.indexing.IndexerResourcePermissionMapper; import org.apache.druid.msq.rpc.ControllerResource; -import org.apache.druid.segment.realtime.firehose.ChatHandler; +import org.apache.druid.segment.realtime.ChatHandler; import org.apache.druid.server.security.AuthorizerMapper; public class ControllerChatHandler extends ControllerResource implements ChatHandler diff --git a/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/indexing/client/WorkerChatHandler.java b/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/indexing/client/WorkerChatHandler.java index 4ef6ab077cac..70d1ab11d380 100644 --- a/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/indexing/client/WorkerChatHandler.java +++ b/extensions-core/multi-stage-query/src/main/java/org/apache/druid/msq/indexing/client/WorkerChatHandler.java @@ -33,8 +33,8 @@ import org.apache.druid.msq.kernel.WorkOrder; import org.apache.druid.msq.statistics.ClusterByStatisticsSnapshot; import org.apache.druid.msq.statistics.serde.ClusterByStatisticsSnapshotSerde; -import org.apache.druid.segment.realtime.firehose.ChatHandler; -import org.apache.druid.segment.realtime.firehose.ChatHandlers; +import org.apache.druid.segment.realtime.ChatHandler; +import org.apache.druid.segment.realtime.ChatHandlers; import org.apache.druid.server.security.Action; import org.apache.druid.utils.CloseableUtils; diff --git a/extensions-core/multi-stage-query/src/test/java/org/apache/druid/msq/test/MSQTestWorkerContext.java b/extensions-core/multi-stage-query/src/test/java/org/apache/druid/msq/test/MSQTestWorkerContext.java index ad05c20b5829..14f6f73b24ab 100644 --- a/extensions-core/multi-stage-query/src/test/java/org/apache/druid/msq/test/MSQTestWorkerContext.java +++ b/extensions-core/multi-stage-query/src/test/java/org/apache/druid/msq/test/MSQTestWorkerContext.java @@ -44,7 +44,7 @@ import org.apache.druid.segment.column.ColumnConfig; import org.apache.druid.segment.incremental.NoopRowIngestionMeters; import org.apache.druid.segment.loading.DataSegmentPusher; -import org.apache.druid.segment.realtime.firehose.NoopChatHandlerProvider; +import org.apache.druid.segment.realtime.NoopChatHandlerProvider; import org.apache.druid.segment.writeout.OffHeapMemorySegmentWriteOutMediumFactory; import org.apache.druid.server.DruidNode; import org.apache.druid.server.coordination.DataSegmentAnnouncer; diff --git a/extensions-core/mysql-metadata-storage/src/main/java/org/apache/druid/firehose/sql/MySQLFirehoseDatabaseConnector.java b/extensions-core/mysql-metadata-storage/src/main/java/org/apache/druid/metadata/input/MySQLInputSourceDatabaseConnector.java similarity index 91% rename from extensions-core/mysql-metadata-storage/src/main/java/org/apache/druid/firehose/sql/MySQLFirehoseDatabaseConnector.java rename to extensions-core/mysql-metadata-storage/src/main/java/org/apache/druid/metadata/input/MySQLInputSourceDatabaseConnector.java index 07f48f36cd39..4647434aba10 100644 --- a/extensions-core/mysql-metadata-storage/src/main/java/org/apache/druid/firehose/sql/MySQLFirehoseDatabaseConnector.java +++ b/extensions-core/mysql-metadata-storage/src/main/java/org/apache/druid/metadata/input/MySQLInputSourceDatabaseConnector.java @@ -17,7 +17,7 @@ * under the License. */ -package org.apache.druid.firehose.sql; +package org.apache.druid.metadata.input; import com.fasterxml.jackson.annotation.JacksonInject; import com.fasterxml.jackson.annotation.JsonCreator; @@ -25,7 +25,7 @@ import com.fasterxml.jackson.annotation.JsonTypeName; import org.apache.commons.dbcp2.BasicDataSource; import org.apache.druid.metadata.MetadataStorageConnectorConfig; -import org.apache.druid.metadata.SQLFirehoseDatabaseConnector; +import org.apache.druid.metadata.SQLInputSourceDatabaseConnector; import org.apache.druid.metadata.storage.mysql.MySQLConnectorDriverConfig; import org.apache.druid.server.initialization.JdbcAccessSecurityConfig; import org.apache.druid.utils.ConnectionUriUtils; @@ -37,7 +37,7 @@ @JsonTypeName("mysql") -public class MySQLFirehoseDatabaseConnector extends SQLFirehoseDatabaseConnector +public class MySQLInputSourceDatabaseConnector extends SQLInputSourceDatabaseConnector { private final DBI dbi; private final MetadataStorageConnectorConfig connectorConfig; @@ -45,7 +45,7 @@ public class MySQLFirehoseDatabaseConnector extends SQLFirehoseDatabaseConnector private final String driverClassName; @JsonCreator - public MySQLFirehoseDatabaseConnector( + public MySQLInputSourceDatabaseConnector( @JsonProperty("connectorConfig") MetadataStorageConnectorConfig connectorConfig, @JsonProperty("driverClassName") @Nullable String driverClassName, @JacksonInject JdbcAccessSecurityConfig securityConfig, @@ -98,7 +98,7 @@ public boolean equals(Object o) if (o == null || getClass() != o.getClass()) { return false; } - MySQLFirehoseDatabaseConnector that = (MySQLFirehoseDatabaseConnector) o; + MySQLInputSourceDatabaseConnector that = (MySQLInputSourceDatabaseConnector) o; return connectorConfig.equals(that.connectorConfig) && Objects.equals( driverClassName, that.driverClassName diff --git a/extensions-core/mysql-metadata-storage/src/main/java/org/apache/druid/metadata/storage/mysql/MySQLMetadataStorageModule.java b/extensions-core/mysql-metadata-storage/src/main/java/org/apache/druid/metadata/storage/mysql/MySQLMetadataStorageModule.java index 2cb8cc7dc79a..3f62e1d7eb36 100644 --- a/extensions-core/mysql-metadata-storage/src/main/java/org/apache/druid/metadata/storage/mysql/MySQLMetadataStorageModule.java +++ b/extensions-core/mysql-metadata-storage/src/main/java/org/apache/druid/metadata/storage/mysql/MySQLMetadataStorageModule.java @@ -24,7 +24,6 @@ import com.fasterxml.jackson.databind.module.SimpleModule; import com.google.inject.Binder; import com.google.inject.Key; -import org.apache.druid.firehose.sql.MySQLFirehoseDatabaseConnector; import org.apache.druid.guice.JsonConfigProvider; import org.apache.druid.guice.LazySingleton; import org.apache.druid.guice.PolyBind; @@ -36,6 +35,7 @@ import org.apache.druid.metadata.MySQLMetadataStorageActionHandlerFactory; import org.apache.druid.metadata.NoopMetadataStorageProvider; import org.apache.druid.metadata.SQLMetadataConnector; +import org.apache.druid.metadata.input.MySQLInputSourceDatabaseConnector; import java.util.Collections; import java.util.List; @@ -55,7 +55,7 @@ public List getJacksonModules() return Collections.singletonList( new SimpleModule() .registerSubtypes( - new NamedType(MySQLFirehoseDatabaseConnector.class, "mysql") + new NamedType(MySQLInputSourceDatabaseConnector.class, "mysql") ) ); } diff --git a/extensions-core/mysql-metadata-storage/src/test/java/org/apache/druid/firehose/sql/MySQLFirehoseDatabaseConnectorTest.java b/extensions-core/mysql-metadata-storage/src/test/java/org/apache/druid/metadata/input/MySQLInputSourceDatabaseConnectorTest.java similarity index 92% rename from extensions-core/mysql-metadata-storage/src/test/java/org/apache/druid/firehose/sql/MySQLFirehoseDatabaseConnectorTest.java rename to extensions-core/mysql-metadata-storage/src/test/java/org/apache/druid/metadata/input/MySQLInputSourceDatabaseConnectorTest.java index 43d96961c1ba..1ed7214aa961 100644 --- a/extensions-core/mysql-metadata-storage/src/test/java/org/apache/druid/firehose/sql/MySQLFirehoseDatabaseConnectorTest.java +++ b/extensions-core/mysql-metadata-storage/src/test/java/org/apache/druid/metadata/input/MySQLInputSourceDatabaseConnectorTest.java @@ -17,7 +17,7 @@ * under the License. */ -package org.apache.druid.firehose.sql; +package org.apache.druid.metadata.input; import com.fasterxml.jackson.core.JsonProcessingException; import com.fasterxml.jackson.databind.InjectableValues; @@ -42,7 +42,7 @@ import java.util.Set; @RunWith(MockitoJUnitRunner.class) -public class MySQLFirehoseDatabaseConnectorTest +public class MySQLInputSourceDatabaseConnectorTest { private static final JdbcAccessSecurityConfig INJECTED_CONF = newSecurityConfigEnforcingAllowList(ImmutableSet.of()); @@ -76,33 +76,33 @@ public String getConnectURI() return "jdbc:mysql://localhost:3306/test"; } }; - MySQLFirehoseDatabaseConnector connector = new MySQLFirehoseDatabaseConnector( + MySQLInputSourceDatabaseConnector connector = new MySQLInputSourceDatabaseConnector( connectorConfig, null, INJECTED_CONF, mySQLConnectorDriverConfig ); - MySQLFirehoseDatabaseConnector andBack = mapper.readValue( + MySQLInputSourceDatabaseConnector andBack = mapper.readValue( mapper.writeValueAsString(connector), - MySQLFirehoseDatabaseConnector.class + MySQLInputSourceDatabaseConnector.class ); Assert.assertEquals(connector, andBack); // test again with classname - connector = new MySQLFirehoseDatabaseConnector( + connector = new MySQLInputSourceDatabaseConnector( connectorConfig, "some.class.name.Driver", INJECTED_CONF, mySQLConnectorDriverConfig ); - andBack = mapper.readValue(mapper.writeValueAsString(connector), MySQLFirehoseDatabaseConnector.class); + andBack = mapper.readValue(mapper.writeValueAsString(connector), MySQLInputSourceDatabaseConnector.class); Assert.assertEquals(connector, andBack); } @Test public void testEqualsAndHashcode() { - EqualsVerifier.forClass(MySQLFirehoseDatabaseConnector.class) + EqualsVerifier.forClass(MySQLInputSourceDatabaseConnector.class) .usingGetClass() .withNonnullFields("connectorConfig") .withIgnoredFields("dbi") @@ -123,7 +123,7 @@ public String getConnectURI() JdbcAccessSecurityConfig securityConfig = newSecurityConfigEnforcingAllowList(ImmutableSet.of()); - new MySQLFirehoseDatabaseConnector( + new MySQLInputSourceDatabaseConnector( connectorConfig, null, securityConfig, @@ -145,7 +145,7 @@ public String getConnectURI() JdbcAccessSecurityConfig securityConfig = newSecurityConfigEnforcingAllowList(ImmutableSet.of("user")); - new MySQLFirehoseDatabaseConnector( + new MySQLInputSourceDatabaseConnector( connectorConfig, null, securityConfig, @@ -170,7 +170,7 @@ public String getConnectURI() expectedException.expectMessage("The property [password] is not in the allowed list"); expectedException.expect(IllegalArgumentException.class); - new MySQLFirehoseDatabaseConnector( + new MySQLInputSourceDatabaseConnector( connectorConfig, null, securityConfig, @@ -194,7 +194,7 @@ public String getConnectURI() ImmutableSet.of("user", "password", "keyonly", "etc") ); - new MySQLFirehoseDatabaseConnector( + new MySQLInputSourceDatabaseConnector( connectorConfig, null, securityConfig, @@ -218,7 +218,7 @@ public String getConnectURI() ImmutableSet.of("user", "password", "keyonly", "etc") ); - new MySQLFirehoseDatabaseConnector( + new MySQLInputSourceDatabaseConnector( connectorConfig, null, securityConfig, @@ -244,7 +244,7 @@ public String getConnectURI() expectedException.expectMessage("The property [password] is not in the allowed list"); expectedException.expect(IllegalArgumentException.class); - new MySQLFirehoseDatabaseConnector( + new MySQLInputSourceDatabaseConnector( connectorConfig, null, securityConfig, @@ -269,7 +269,7 @@ public String getConnectURI() expectedException.expectMessage("The property [password] is not in the allowed list"); expectedException.expect(IllegalArgumentException.class); - new MySQLFirehoseDatabaseConnector( + new MySQLInputSourceDatabaseConnector( connectorConfig, null, securityConfig, @@ -294,7 +294,7 @@ public String getConnectURI() expectedException.expectMessage("The property [password] is not in the allowed list"); expectedException.expect(IllegalArgumentException.class); - new MySQLFirehoseDatabaseConnector( + new MySQLInputSourceDatabaseConnector( connectorConfig, null, securityConfig, @@ -329,7 +329,7 @@ public boolean isEnforceAllowedProperties() } }; - new MySQLFirehoseDatabaseConnector( + new MySQLInputSourceDatabaseConnector( connectorConfig, null, securityConfig, diff --git a/extensions-core/postgresql-metadata-storage/src/main/java/org/apache/druid/firehose/PostgresqlFirehoseDatabaseConnector.java b/extensions-core/postgresql-metadata-storage/src/main/java/org/apache/druid/metadata/input/PostgresqlInputSourceDatabaseConnector.java similarity index 87% rename from extensions-core/postgresql-metadata-storage/src/main/java/org/apache/druid/firehose/PostgresqlFirehoseDatabaseConnector.java rename to extensions-core/postgresql-metadata-storage/src/main/java/org/apache/druid/metadata/input/PostgresqlInputSourceDatabaseConnector.java index d9880d7acf2f..9812b213f59f 100644 --- a/extensions-core/postgresql-metadata-storage/src/main/java/org/apache/druid/firehose/PostgresqlFirehoseDatabaseConnector.java +++ b/extensions-core/postgresql-metadata-storage/src/main/java/org/apache/druid/metadata/input/PostgresqlInputSourceDatabaseConnector.java @@ -17,7 +17,7 @@ * under the License. */ -package org.apache.druid.firehose; +package org.apache.druid.metadata.input; import com.fasterxml.jackson.annotation.JacksonInject; import com.fasterxml.jackson.annotation.JsonCreator; @@ -25,7 +25,7 @@ import com.fasterxml.jackson.annotation.JsonTypeName; import org.apache.commons.dbcp2.BasicDataSource; import org.apache.druid.metadata.MetadataStorageConnectorConfig; -import org.apache.druid.metadata.SQLFirehoseDatabaseConnector; +import org.apache.druid.metadata.SQLInputSourceDatabaseConnector; import org.apache.druid.server.initialization.JdbcAccessSecurityConfig; import org.apache.druid.utils.ConnectionUriUtils; import org.skife.jdbi.v2.DBI; @@ -35,13 +35,13 @@ @JsonTypeName("postgresql") -public class PostgresqlFirehoseDatabaseConnector extends SQLFirehoseDatabaseConnector +public class PostgresqlInputSourceDatabaseConnector extends SQLInputSourceDatabaseConnector { private final DBI dbi; private final MetadataStorageConnectorConfig connectorConfig; @JsonCreator - public PostgresqlFirehoseDatabaseConnector( + public PostgresqlInputSourceDatabaseConnector( @JsonProperty("connectorConfig") MetadataStorageConnectorConfig connectorConfig, @JacksonInject JdbcAccessSecurityConfig securityConfig ) @@ -80,7 +80,7 @@ public boolean equals(Object o) if (o == null || getClass() != o.getClass()) { return false; } - PostgresqlFirehoseDatabaseConnector that = (PostgresqlFirehoseDatabaseConnector) o; + PostgresqlInputSourceDatabaseConnector that = (PostgresqlInputSourceDatabaseConnector) o; return connectorConfig.equals(that.connectorConfig); } @@ -93,7 +93,7 @@ public int hashCode() @Override public String toString() { - return "PostgresqlFirehoseDatabaseConnector{" + + return "PostgresqlInputSourceDatabaseConnector{" + "connectorConfig=" + connectorConfig + '}'; } diff --git a/extensions-core/postgresql-metadata-storage/src/main/java/org/apache/druid/metadata/storage/postgresql/PostgreSQLMetadataStorageModule.java b/extensions-core/postgresql-metadata-storage/src/main/java/org/apache/druid/metadata/storage/postgresql/PostgreSQLMetadataStorageModule.java index 9506eddd04bf..31777d0f4314 100644 --- a/extensions-core/postgresql-metadata-storage/src/main/java/org/apache/druid/metadata/storage/postgresql/PostgreSQLMetadataStorageModule.java +++ b/extensions-core/postgresql-metadata-storage/src/main/java/org/apache/druid/metadata/storage/postgresql/PostgreSQLMetadataStorageModule.java @@ -24,7 +24,6 @@ import com.fasterxml.jackson.databind.module.SimpleModule; import com.google.inject.Binder; import com.google.inject.Key; -import org.apache.druid.firehose.PostgresqlFirehoseDatabaseConnector; import org.apache.druid.guice.JsonConfigProvider; import org.apache.druid.guice.LazySingleton; import org.apache.druid.guice.PolyBind; @@ -36,6 +35,7 @@ import org.apache.druid.metadata.NoopMetadataStorageProvider; import org.apache.druid.metadata.PostgreSQLMetadataStorageActionHandlerFactory; import org.apache.druid.metadata.SQLMetadataConnector; +import org.apache.druid.metadata.input.PostgresqlInputSourceDatabaseConnector; import java.util.Collections; import java.util.List; @@ -56,7 +56,7 @@ public List getJacksonModules() return Collections.singletonList( new SimpleModule() .registerSubtypes( - new NamedType(PostgresqlFirehoseDatabaseConnector.class, "postgresql") + new NamedType(PostgresqlInputSourceDatabaseConnector.class, "postgresql") ) ); } diff --git a/extensions-core/postgresql-metadata-storage/src/test/java/org/apache/druid/firehose/PostgresqlFirehoseDatabaseConnectorTest.java b/extensions-core/postgresql-metadata-storage/src/test/java/org/apache/druid/metadata/input/PostgresqlInputSourceDatabaseConnectorTest.java similarity index 90% rename from extensions-core/postgresql-metadata-storage/src/test/java/org/apache/druid/firehose/PostgresqlFirehoseDatabaseConnectorTest.java rename to extensions-core/postgresql-metadata-storage/src/test/java/org/apache/druid/metadata/input/PostgresqlInputSourceDatabaseConnectorTest.java index 9b93f0102c45..0d67d72aa848 100644 --- a/extensions-core/postgresql-metadata-storage/src/test/java/org/apache/druid/firehose/PostgresqlFirehoseDatabaseConnectorTest.java +++ b/extensions-core/postgresql-metadata-storage/src/test/java/org/apache/druid/metadata/input/PostgresqlInputSourceDatabaseConnectorTest.java @@ -17,7 +17,7 @@ * under the License. */ -package org.apache.druid.firehose; +package org.apache.druid.metadata.input; import com.fasterxml.jackson.core.JsonProcessingException; import com.fasterxml.jackson.databind.InjectableValues; @@ -35,7 +35,7 @@ import java.util.Set; -public class PostgresqlFirehoseDatabaseConnectorTest +public class PostgresqlInputSourceDatabaseConnectorTest { private static final ObjectMapper MAPPER = new DefaultObjectMapper(); private static final JdbcAccessSecurityConfig INJECTED_CONF = newSecurityConfigEnforcingAllowList(ImmutableSet.of()); @@ -60,13 +60,13 @@ public String getConnectURI() return "jdbc:postgresql://localhost:3306/test"; } }; - PostgresqlFirehoseDatabaseConnector connector = new PostgresqlFirehoseDatabaseConnector( + PostgresqlInputSourceDatabaseConnector connector = new PostgresqlInputSourceDatabaseConnector( connectorConfig, INJECTED_CONF ); - PostgresqlFirehoseDatabaseConnector andBack = MAPPER.readValue( + PostgresqlInputSourceDatabaseConnector andBack = MAPPER.readValue( MAPPER.writeValueAsString(connector), - PostgresqlFirehoseDatabaseConnector.class + PostgresqlInputSourceDatabaseConnector.class ); Assert.assertEquals(connector, andBack); } @@ -74,7 +74,7 @@ public String getConnectURI() @Test public void testEqualsAndHashcode() { - EqualsVerifier.forClass(PostgresqlFirehoseDatabaseConnector.class) + EqualsVerifier.forClass(PostgresqlInputSourceDatabaseConnector.class) .usingGetClass() .withNonnullFields("connectorConfig") .withIgnoredFields("dbi") @@ -95,7 +95,7 @@ public String getConnectURI() JdbcAccessSecurityConfig securityConfig = newSecurityConfigEnforcingAllowList(ImmutableSet.of()); - new PostgresqlFirehoseDatabaseConnector( + new PostgresqlInputSourceDatabaseConnector( connectorConfig, securityConfig ); @@ -115,7 +115,7 @@ public String getConnectURI() JdbcAccessSecurityConfig securityConfig = newSecurityConfigEnforcingAllowList(ImmutableSet.of("user")); - new PostgresqlFirehoseDatabaseConnector( + new PostgresqlInputSourceDatabaseConnector( connectorConfig, securityConfig ); @@ -138,7 +138,7 @@ public String getConnectURI() expectedException.expectMessage("is not in the allowed list"); expectedException.expect(IllegalArgumentException.class); - new PostgresqlFirehoseDatabaseConnector( + new PostgresqlInputSourceDatabaseConnector( connectorConfig, securityConfig ); @@ -160,7 +160,7 @@ public String getConnectURI() ImmutableSet.of("user", "password", "keyonly", "etc") ); - new PostgresqlFirehoseDatabaseConnector( + new PostgresqlInputSourceDatabaseConnector( connectorConfig, securityConfig ); @@ -183,7 +183,7 @@ public String getConnectURI() expectedException.expectMessage("is not in the allowed list"); expectedException.expect(IllegalArgumentException.class); - new PostgresqlFirehoseDatabaseConnector( + new PostgresqlInputSourceDatabaseConnector( connectorConfig, securityConfig ); @@ -206,7 +206,7 @@ public String getConnectURI() expectedException.expectMessage("is not in the allowed list"); expectedException.expect(IllegalArgumentException.class); - new PostgresqlFirehoseDatabaseConnector( + new PostgresqlInputSourceDatabaseConnector( connectorConfig, securityConfig ); @@ -239,7 +239,7 @@ public boolean isEnforceAllowedProperties() } }; - new PostgresqlFirehoseDatabaseConnector( + new PostgresqlInputSourceDatabaseConnector( connectorConfig, securityConfig ); diff --git a/indexing-hadoop/src/main/java/org/apache/druid/indexer/hadoop/DatasourceRecordReader.java b/indexing-hadoop/src/main/java/org/apache/druid/indexer/hadoop/DatasourceRecordReader.java index 1e5da8af8ed6..40a67f1236e0 100644 --- a/indexing-hadoop/src/main/java/org/apache/druid/indexer/hadoop/DatasourceRecordReader.java +++ b/indexing-hadoop/src/main/java/org/apache/druid/indexer/hadoop/DatasourceRecordReader.java @@ -20,36 +20,62 @@ package org.apache.druid.indexer.hadoop; import com.google.common.base.Function; +import com.google.common.collect.Iterables; import com.google.common.collect.Iterators; import com.google.common.collect.Lists; +import com.google.common.collect.Maps; import com.google.common.io.Closeables; import org.apache.druid.data.input.InputRow; +import org.apache.druid.data.input.MapBasedInputRow; import org.apache.druid.data.input.Row; +import org.apache.druid.data.input.impl.TimestampSpec; import org.apache.druid.indexer.HadoopDruidIndexerConfig; import org.apache.druid.indexer.JobHelper; +import org.apache.druid.java.util.common.DateTimes; import org.apache.druid.java.util.common.FileUtils; +import org.apache.druid.java.util.common.granularity.Granularities; +import org.apache.druid.java.util.common.guava.Sequence; +import org.apache.druid.java.util.common.guava.Sequences; +import org.apache.druid.java.util.common.guava.Yielder; +import org.apache.druid.java.util.common.guava.Yielders; import org.apache.druid.java.util.common.logger.Logger; +import org.apache.druid.query.dimension.DefaultDimensionSpec; +import org.apache.druid.query.filter.DimFilter; +import org.apache.druid.segment.BaseLongColumnValueSelector; +import org.apache.druid.segment.BaseObjectColumnValueSelector; +import org.apache.druid.segment.Cursor; +import org.apache.druid.segment.DimensionSelector; import org.apache.druid.segment.QueryableIndex; import org.apache.druid.segment.QueryableIndexStorageAdapter; -import org.apache.druid.segment.realtime.firehose.IngestSegmentFirehose; -import org.apache.druid.segment.realtime.firehose.WindowedStorageAdapter; +import org.apache.druid.segment.VirtualColumns; +import org.apache.druid.segment.column.ColumnHolder; +import org.apache.druid.segment.data.IndexedInts; +import org.apache.druid.segment.filter.Filters; +import org.apache.druid.segment.realtime.WindowedStorageAdapter; +import org.apache.druid.segment.transform.TransformSpec; +import org.apache.druid.segment.transform.Transformer; import org.apache.hadoop.fs.Path; import org.apache.hadoop.io.NullWritable; import org.apache.hadoop.mapreduce.InputSplit; import org.apache.hadoop.mapreduce.RecordReader; import org.apache.hadoop.mapreduce.TaskAttemptContext; +import javax.annotation.Nullable; +import java.io.Closeable; import java.io.File; import java.io.IOException; import java.util.ArrayList; +import java.util.HashMap; +import java.util.Iterator; import java.util.List; +import java.util.Map; public class DatasourceRecordReader extends RecordReader { private static final Logger logger = new Logger(DatasourceRecordReader.class); private DatasourceIngestionSpec spec; - private IngestSegmentFirehose firehose; + private SegmentReader segmentReader; private long rowNum; private Row currRow; @@ -108,7 +134,7 @@ public WindowedStorageAdapter apply(WindowedDataSegment segment) } ); - firehose = new IngestSegmentFirehose( + segmentReader = new SegmentReader( adapters, spec.getTransformSpec(), spec.getDimensions(), @@ -120,8 +146,8 @@ public WindowedStorageAdapter apply(WindowedDataSegment segment) @Override public boolean nextKeyValue() { - if (firehose.hasMore()) { - currRow = firehose.nextRow(); + if (segmentReader.hasMore()) { + currRow = segmentReader.nextRow(); rowNum++; return true; } else { @@ -154,7 +180,7 @@ public float getProgress() @Override public void close() throws IOException { - Closeables.close(firehose, true); + Closeables.close(segmentReader, true); for (QueryableIndex qi : indexes) { Closeables.close(qi, true); } @@ -163,4 +189,157 @@ public void close() throws IOException FileUtils.deleteDirectory(dir); } } + + public static class SegmentReader implements Closeable + { + private final Transformer transformer; + private Yielder rowYielder; + + public SegmentReader( + final List adapters, + final TransformSpec transformSpec, + final List dims, + final List metrics, + final DimFilter dimFilter + ) + { + this.transformer = transformSpec.toTransformer(); + + Sequence rows = Sequences.concat( + Iterables.transform( + adapters, + new Function>() + { + @Nullable + @Override + public Sequence apply(WindowedStorageAdapter adapter) + { + return Sequences.concat( + Sequences.map( + adapter.getAdapter().makeCursors( + Filters.toFilter(dimFilter), + adapter.getInterval(), + VirtualColumns.EMPTY, + Granularities.ALL, + false, + null + ), new Function>() + { + @Nullable + @Override + public Sequence apply(final Cursor cursor) + { + final BaseLongColumnValueSelector timestampColumnSelector = + cursor.getColumnSelectorFactory().makeColumnValueSelector(ColumnHolder.TIME_COLUMN_NAME); + + final Map dimSelectors = new HashMap<>(); + for (String dim : dims) { + final DimensionSelector dimSelector = cursor + .getColumnSelectorFactory() + .makeDimensionSelector(new DefaultDimensionSpec(dim, dim)); + // dimSelector is null if the dimension is not present + if (dimSelector != null) { + dimSelectors.put(dim, dimSelector); + } + } + + final Map metSelectors = new HashMap<>(); + for (String metric : metrics) { + final BaseObjectColumnValueSelector metricSelector = + cursor.getColumnSelectorFactory().makeColumnValueSelector(metric); + metSelectors.put(metric, metricSelector); + } + + return Sequences.simple( + new Iterable() + { + @Override + public Iterator iterator() + { + return new Iterator() + { + @Override + public boolean hasNext() + { + return !cursor.isDone(); + } + + @Override + public InputRow next() + { + final Map theEvent = Maps.newLinkedHashMap(); + final long timestamp = timestampColumnSelector.getLong(); + theEvent.put(TimestampSpec.DEFAULT_COLUMN, DateTimes.utc(timestamp)); + + for (Map.Entry dimSelector : + dimSelectors.entrySet()) { + final String dim = dimSelector.getKey(); + final DimensionSelector selector = dimSelector.getValue(); + final IndexedInts vals = selector.getRow(); + + int valsSize = vals.size(); + if (valsSize == 1) { + final String dimVal = selector.lookupName(vals.get(0)); + theEvent.put(dim, dimVal); + } else if (valsSize > 1) { + List dimVals = new ArrayList<>(valsSize); + for (int i = 0; i < valsSize; ++i) { + dimVals.add(selector.lookupName(vals.get(i))); + } + theEvent.put(dim, dimVals); + } + } + + for (Map.Entry metSelector : + metSelectors.entrySet()) { + final String metric = metSelector.getKey(); + final BaseObjectColumnValueSelector selector = metSelector.getValue(); + Object value = selector.getObject(); + if (value != null) { + theEvent.put(metric, value); + } + } + cursor.advance(); + return new MapBasedInputRow(timestamp, dims, theEvent); + } + + @Override + public void remove() + { + throw new UnsupportedOperationException("Remove Not Supported"); + } + }; + } + } + ); + } + } + ) + ); + } + } + ) + ); + rowYielder = Yielders.each(rows); + } + + public boolean hasMore() + { + return !rowYielder.isDone(); + } + + @Nullable + public InputRow nextRow() + { + final InputRow inputRow = rowYielder.get(); + rowYielder = rowYielder.next(null); + return transformer.transform(inputRow); + } + + @Override + public void close() throws IOException + { + rowYielder.close(); + } + } } diff --git a/indexing-hadoop/src/test/java/org/apache/druid/indexer/BatchDeltaIngestionTest.java b/indexing-hadoop/src/test/java/org/apache/druid/indexer/BatchDeltaIngestionTest.java index ed8b8c0bb093..6516b0a0e00d 100644 --- a/indexing-hadoop/src/test/java/org/apache/druid/indexer/BatchDeltaIngestionTest.java +++ b/indexing-hadoop/src/test/java/org/apache/druid/indexer/BatchDeltaIngestionTest.java @@ -26,13 +26,13 @@ import com.google.common.collect.ImmutableMap; import com.google.common.collect.Iterables; import org.apache.commons.io.FileUtils; -import org.apache.druid.data.input.Firehose; import org.apache.druid.data.input.InputRow; import org.apache.druid.data.input.impl.CSVParseSpec; import org.apache.druid.data.input.impl.DimensionsSpec; import org.apache.druid.data.input.impl.StringInputRowParser; import org.apache.druid.data.input.impl.TimestampSpec; import org.apache.druid.hll.HyperLogLogCollector; +import org.apache.druid.indexer.hadoop.DatasourceRecordReader; import org.apache.druid.indexer.hadoop.WindowedDataSegment; import org.apache.druid.jackson.DefaultObjectMapper; import org.apache.druid.java.util.common.DateTimes; @@ -49,8 +49,7 @@ import org.apache.druid.segment.indexing.DataSchema; import org.apache.druid.segment.indexing.granularity.UniformGranularitySpec; import org.apache.druid.segment.loading.LocalDataSegmentPuller; -import org.apache.druid.segment.realtime.firehose.IngestSegmentFirehose; -import org.apache.druid.segment.realtime.firehose.WindowedStorageAdapter; +import org.apache.druid.segment.realtime.WindowedStorageAdapter; import org.apache.druid.segment.transform.TransformSpec; import org.apache.druid.timeline.DataSegment; import org.apache.druid.timeline.DataSegment.PruneSpecsHolder; @@ -404,7 +403,7 @@ private void testIngestion( QueryableIndex index = INDEX_IO.loadIndex(tmpUnzippedSegmentDir); StorageAdapter adapter = new QueryableIndexStorageAdapter(index); - Firehose firehose = new IngestSegmentFirehose( + DatasourceRecordReader.SegmentReader segmentReader = new DatasourceRecordReader.SegmentReader( ImmutableList.of(new WindowedStorageAdapter(adapter, windowedDataSegment.getInterval())), TransformSpec.NONE, expectedDimensions, @@ -413,11 +412,12 @@ private void testIngestion( ); List rows = new ArrayList<>(); - while (firehose.hasMore()) { - rows.add(firehose.nextRow()); + while (segmentReader.hasMore()) { + rows.add(segmentReader.nextRow()); } verifyRows(expectedRowsGenerated, rows, expectedDimensions, expectedMetrics); + segmentReader.close(); } private HadoopDruidIndexerConfig makeHadoopDruidIndexerConfig(Map inputSpec, File tmpDir) diff --git a/server/src/test/java/org/apache/druid/segment/realtime/firehose/IngestSegmentFirehoseTest.java b/indexing-hadoop/src/test/java/org/apache/druid/indexer/hadoop/DatasourceRecordReaderSegmentReaderTest.java similarity index 92% rename from server/src/test/java/org/apache/druid/segment/realtime/firehose/IngestSegmentFirehoseTest.java rename to indexing-hadoop/src/test/java/org/apache/druid/indexer/hadoop/DatasourceRecordReaderSegmentReaderTest.java index 44f24f879a22..042aa5e9550f 100644 --- a/server/src/test/java/org/apache/druid/segment/realtime/firehose/IngestSegmentFirehoseTest.java +++ b/indexing-hadoop/src/test/java/org/apache/druid/indexer/hadoop/DatasourceRecordReaderSegmentReaderTest.java @@ -17,7 +17,7 @@ * under the License. */ -package org.apache.druid.segment.realtime.firehose; +package org.apache.druid.indexer.hadoop; import com.google.common.collect.ImmutableList; import com.google.common.collect.Lists; @@ -47,6 +47,7 @@ import org.apache.druid.segment.incremental.IncrementalIndexSchema; import org.apache.druid.segment.incremental.IncrementalIndexStorageAdapter; import org.apache.druid.segment.incremental.OnheapIncrementalIndex; +import org.apache.druid.segment.realtime.WindowedStorageAdapter; import org.apache.druid.segment.transform.TransformSpec; import org.apache.druid.segment.writeout.OffHeapMemorySegmentWriteOutMediumFactory; import org.apache.druid.segment.writeout.SegmentWriteOutMediumFactory; @@ -66,7 +67,7 @@ /** */ @RunWith(Parameterized.class) -public class IngestSegmentFirehoseTest +public class DatasourceRecordReaderSegmentReaderTest { private static final DimensionsSpec DIMENSIONS_SPEC = new DimensionsSpec( ImmutableList.of( @@ -107,7 +108,7 @@ public static Collection constructorFeeder() private final IndexIO indexIO; private final IndexMerger indexMerger; - public IngestSegmentFirehoseTest(SegmentWriteOutMediumFactory segmentWriteOutMediumFactory) + public DatasourceRecordReaderSegmentReaderTest(SegmentWriteOutMediumFactory segmentWriteOutMediumFactory) { indexIO = TestHelper.getTestIndexIO(); indexMerger = TestHelper.getTestIndexMergerV9(segmentWriteOutMediumFactory); @@ -135,7 +136,7 @@ public void testReadFromIndexAndWriteAnotherIndex() throws Exception ) { final StorageAdapter sa = new QueryableIndexStorageAdapter(qi); final WindowedStorageAdapter wsa = new WindowedStorageAdapter(sa, sa.getInterval()); - final IngestSegmentFirehose firehose = new IngestSegmentFirehose( + final DatasourceRecordReader.SegmentReader segmentReader = new DatasourceRecordReader.SegmentReader( ImmutableList.of(wsa, wsa), TransformSpec.NONE, ImmutableList.of("host", "spatial"), @@ -144,8 +145,8 @@ public void testReadFromIndexAndWriteAnotherIndex() throws Exception ); int count = 0; - while (firehose.hasMore()) { - final InputRow row = firehose.nextRow(); + while (segmentReader.hasMore()) { + final InputRow row = segmentReader.nextRow(); Assert.assertNotNull(row); if (count == 0) { Assert.assertEquals(DateTimes.of("2014-10-22T00Z"), row.getTimestamp()); @@ -168,15 +169,15 @@ public void testReadFromIndexAndWriteAnotherIndex() throws Exception Assert.assertEquals(ImmutableList.of("visited_sum", "unique_hosts"), queryable.getAvailableMetrics()); // Do a spatial filter - final IngestSegmentFirehose firehose2 = new IngestSegmentFirehose( + final DatasourceRecordReader.SegmentReader segmentReader2 = new DatasourceRecordReader.SegmentReader( ImmutableList.of(new WindowedStorageAdapter(queryable, Intervals.of("2000/3000"))), TransformSpec.NONE, ImmutableList.of("host", "spatial"), ImmutableList.of("visited_sum", "unique_hosts"), new SpatialDimFilter("spatial", new RadiusBound(new float[]{1, 0}, 0.1f)) ); - final InputRow row = firehose2.nextRow(); - Assert.assertFalse(firehose2.hasMore()); + final InputRow row = segmentReader2.nextRow(); + Assert.assertFalse(segmentReader2.hasMore()); Assert.assertEquals(DateTimes.of("2014-10-22T00Z"), row.getTimestamp()); Assert.assertEquals("host2", row.getRaw("host")); Assert.assertEquals("1,0", row.getRaw("spatial")); diff --git a/indexing-service/src/main/java/org/apache/druid/guice/IndexingServiceFirehoseModule.java b/indexing-service/src/main/java/org/apache/druid/guice/IndexingServiceFirehoseModule.java deleted file mode 100644 index 8e4952210997..000000000000 --- a/indexing-service/src/main/java/org/apache/druid/guice/IndexingServiceFirehoseModule.java +++ /dev/null @@ -1,49 +0,0 @@ -/* - * Licensed to the Apache Software Foundation (ASF) under one - * or more contributor license agreements. See the NOTICE file - * distributed with this work for additional information - * regarding copyright ownership. The ASF licenses this file - * to you under the Apache License, Version 2.0 (the - * "License"); you may not use this file except in compliance - * with the License. You may obtain a copy of the License at - * - * http://www.apache.org/licenses/LICENSE-2.0 - * - * Unless required by applicable law or agreed to in writing, - * software distributed under the License is distributed on an - * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY - * KIND, either express or implied. See the License for the - * specific language governing permissions and limitations - * under the License. - */ - -package org.apache.druid.guice; - -import com.fasterxml.jackson.databind.Module; -import com.fasterxml.jackson.databind.jsontype.NamedType; -import com.fasterxml.jackson.databind.module.SimpleModule; -import com.google.common.collect.ImmutableList; -import com.google.inject.Binder; -import org.apache.druid.initialization.DruidModule; -import org.apache.druid.segment.realtime.firehose.EventReceiverFirehoseFactory; - -import java.util.List; - -public class IndexingServiceFirehoseModule implements DruidModule -{ - @Override - public List getJacksonModules() - { - return ImmutableList.of( - new SimpleModule("IndexingServiceFirehoseModule") - .registerSubtypes( - new NamedType(EventReceiverFirehoseFactory.class, "receiver") - ) - ); - } - - @Override - public void configure(Binder binder) - { - } -} diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/common/TaskToolbox.java b/indexing-service/src/main/java/org/apache/druid/indexing/common/TaskToolbox.java index 46de3064f033..9732d72eaf5e 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/common/TaskToolbox.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/common/TaskToolbox.java @@ -58,9 +58,9 @@ import org.apache.druid.segment.loading.SegmentCacheManager; import org.apache.druid.segment.loading.SegmentLoaderConfig; import org.apache.druid.segment.metadata.CentralizedDatasourceSchemaConfig; +import org.apache.druid.segment.realtime.ChatHandlerProvider; import org.apache.druid.segment.realtime.appenderator.AppenderatorsManager; import org.apache.druid.segment.realtime.appenderator.UnifiedIndexerAppenderatorsManager; -import org.apache.druid.segment.realtime.firehose.ChatHandlerProvider; import org.apache.druid.server.DruidNode; import org.apache.druid.server.coordination.DataSegmentAnnouncer; import org.apache.druid.server.coordination.DataSegmentServerAnnouncer; diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/common/TaskToolboxFactory.java b/indexing-service/src/main/java/org/apache/druid/indexing/common/TaskToolboxFactory.java index 669a30dc5de9..7190c38849f2 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/common/TaskToolboxFactory.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/common/TaskToolboxFactory.java @@ -59,8 +59,8 @@ import org.apache.druid.segment.loading.DataSegmentPusher; import org.apache.druid.segment.loading.SegmentLoaderConfig; import org.apache.druid.segment.metadata.CentralizedDatasourceSchemaConfig; +import org.apache.druid.segment.realtime.ChatHandlerProvider; import org.apache.druid.segment.realtime.appenderator.AppenderatorsManager; -import org.apache.druid.segment.realtime.firehose.ChatHandlerProvider; import org.apache.druid.server.DruidNode; import org.apache.druid.server.coordination.DataSegmentAnnouncer; import org.apache.druid.server.coordination.DataSegmentServerAnnouncer; diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/common/task/HadoopIndexTask.java b/indexing-service/src/main/java/org/apache/druid/indexing/common/task/HadoopIndexTask.java index bc9d5aca2b65..f1f96d8ca34d 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/common/task/HadoopIndexTask.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/common/task/HadoopIndexTask.java @@ -59,8 +59,8 @@ import org.apache.druid.segment.incremental.RowIngestionMeters; import org.apache.druid.segment.indexing.granularity.ArbitraryGranularitySpec; import org.apache.druid.segment.indexing.granularity.GranularitySpec; -import org.apache.druid.segment.realtime.firehose.ChatHandler; -import org.apache.druid.segment.realtime.firehose.ChatHandlerProvider; +import org.apache.druid.segment.realtime.ChatHandler; +import org.apache.druid.segment.realtime.ChatHandlerProvider; import org.apache.druid.server.security.Action; import org.apache.druid.server.security.AuthorizerMapper; import org.apache.druid.server.security.Resource; diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/common/task/IndexTask.java b/indexing-service/src/main/java/org/apache/druid/indexing/common/task/IndexTask.java index dc6c07b6b83c..a8f60cca5729 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/common/task/IndexTask.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/common/task/IndexTask.java @@ -21,8 +21,6 @@ import com.fasterxml.jackson.annotation.JsonCreator; import com.fasterxml.jackson.annotation.JsonIgnore; -import com.fasterxml.jackson.annotation.JsonInclude; -import com.fasterxml.jackson.annotation.JsonInclude.Include; import com.fasterxml.jackson.annotation.JsonProperty; import com.fasterxml.jackson.annotation.JsonTypeName; import com.fasterxml.jackson.databind.ObjectMapper; @@ -34,7 +32,6 @@ import com.google.common.hash.HashFunction; import com.google.common.hash.Hashing; import com.google.common.util.concurrent.ListenableFuture; -import org.apache.druid.data.input.FirehoseFactory; import org.apache.druid.data.input.InputFormat; import org.apache.druid.data.input.InputRow; import org.apache.druid.data.input.InputSource; @@ -87,6 +84,7 @@ import org.apache.druid.segment.indexing.TuningConfig; import org.apache.druid.segment.indexing.granularity.ArbitraryGranularitySpec; import org.apache.druid.segment.indexing.granularity.GranularitySpec; +import org.apache.druid.segment.realtime.ChatHandler; import org.apache.druid.segment.realtime.SegmentGenerationMetrics; import org.apache.druid.segment.realtime.appenderator.Appenderator; import org.apache.druid.segment.realtime.appenderator.AppenderatorConfig; @@ -95,7 +93,6 @@ import org.apache.druid.segment.realtime.appenderator.SegmentIdWithShardSpec; import org.apache.druid.segment.realtime.appenderator.SegmentsAndCommitMetadata; import org.apache.druid.segment.realtime.appenderator.TransactionalSegmentPublisher; -import org.apache.druid.segment.realtime.firehose.ChatHandler; import org.apache.druid.segment.writeout.SegmentWriteOutMediumFactory; import org.apache.druid.server.security.Action; import org.apache.druid.server.security.AuthorizerMapper; @@ -312,9 +309,6 @@ public String getTaskAllocatorId() @Override public Set getInputSourceResources() { - if (ingestionSchema.getIOConfig().firehoseFactory != null) { - throw getInputSecurityOnFirehoseUnsupportedError(); - } return getIngestionSchema().getIOConfig().getInputSource() != null ? getIngestionSchema().getIOConfig().getInputSource().getTypes() .stream() @@ -1123,8 +1117,6 @@ public IndexTuningConfig getTuningConfig() @JsonTypeName("index") public static class IndexIOConfig implements BatchIOConfig { - - private final FirehoseFactory firehoseFactory; private final InputSource inputSource; private final AtomicReference inputSourceWithToolbox = new AtomicReference<>(); private final InputFormat inputFormat; @@ -1133,42 +1125,18 @@ public static class IndexIOConfig implements BatchIOConfig @JsonCreator public IndexIOConfig( - @Deprecated @JsonProperty("firehose") @Nullable FirehoseFactory firehoseFactory, @JsonProperty("inputSource") @Nullable InputSource inputSource, @JsonProperty("inputFormat") @Nullable InputFormat inputFormat, @JsonProperty("appendToExisting") @Nullable Boolean appendToExisting, @JsonProperty("dropExisting") @Nullable Boolean dropExisting ) { - Checks.checkOneNotNullOrEmpty( - ImmutableList.of(new Property<>("firehose", firehoseFactory), new Property<>("inputSource", inputSource)) - ); - if (firehoseFactory != null && inputFormat != null) { - throw new IAE("Cannot use firehose and inputFormat together. Try using inputSource instead of firehose."); - } - this.firehoseFactory = firehoseFactory; this.inputSource = inputSource; this.inputFormat = inputFormat; this.appendToExisting = appendToExisting == null ? BatchIOConfig.DEFAULT_APPEND_EXISTING : appendToExisting; this.dropExisting = dropExisting == null ? BatchIOConfig.DEFAULT_DROP_EXISTING : dropExisting; } - // old constructor for backward compatibility - @Deprecated - public IndexIOConfig(FirehoseFactory firehoseFactory, @Nullable Boolean appendToExisting, @Nullable Boolean dropExisting) - { - this(firehoseFactory, null, null, appendToExisting, dropExisting); - } - - @Nullable - @JsonProperty("firehose") - @JsonInclude(Include.NON_NULL) - @Deprecated - public FirehoseFactory getFirehoseFactory() - { - return firehoseFactory; - } - @Nullable @Override @JsonProperty diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/common/task/NativeCompactionRunner.java b/indexing-service/src/main/java/org/apache/druid/indexing/common/task/NativeCompactionRunner.java index f2eacb8c1c6c..30761b674e54 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/common/task/NativeCompactionRunner.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/common/task/NativeCompactionRunner.java @@ -157,7 +157,6 @@ private static ParallelIndexIOConfig createIoConfig( } return new ParallelIndexIOConfig( - null, new DruidInputSource( dataSchema.getDataSource(), interval, diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/common/task/Task.java b/indexing-service/src/main/java/org/apache/druid/indexing/common/task/Task.java index 4fb4bb7852c0..9b882e2e8d2b 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/common/task/Task.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/common/task/Task.java @@ -152,8 +152,7 @@ default int getPriority() * the task does not use any. Users can be given permission to access particular types of * input sources but not others, using the * {@link org.apache.druid.server.security.AuthConfig#enableInputSourceSecurity} config. - * @throws UnsupportedOperationException if the given task type does not suppoert input source based security. Such - * would be the case, if the task uses firehose. + * @throws UnsupportedOperationException if the given task type does not suppoert input source based security */ @JsonIgnore @Nonnull @@ -165,15 +164,6 @@ default Set getInputSourceResources() throws UOE )); } - default UOE getInputSecurityOnFirehoseUnsupportedError() - { - throw new UOE(StringUtils.format( - "Input source based security cannot be performed '%s' task because it uses firehose." - + " Change the tasks configuration, or disable `isEnableInputSourceSecurity`", - getType() - )); - } - /** * Returns query runners for this task. If this task is not meant to answer queries over its datasource, this method * should return null. diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/common/task/batch/parallel/InputSourceSplitParallelIndexTaskRunner.java b/indexing-service/src/main/java/org/apache/druid/indexing/common/task/batch/parallel/InputSourceSplitParallelIndexTaskRunner.java index 286325186b25..e6097678b5ae 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/common/task/batch/parallel/InputSourceSplitParallelIndexTaskRunner.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/common/task/batch/parallel/InputSourceSplitParallelIndexTaskRunner.java @@ -19,7 +19,6 @@ package org.apache.druid.indexing.common.task.batch.parallel; -import org.apache.druid.data.input.FirehoseFactory; import org.apache.druid.data.input.InputSource; import org.apache.druid.data.input.InputSplit; import org.apache.druid.data.input.impl.SplittableInputSource; @@ -81,14 +80,11 @@ final int estimateTotalNumSubTasks() throws IOException final SubTaskSpec newTaskSpec(InputSplit split) { - final FirehoseFactory firehoseFactory; final InputSource inputSource; - firehoseFactory = null; inputSource = baseInputSource.withSplit(split); final ParallelIndexIngestionSpec subTaskIngestionSpec = new ParallelIndexIngestionSpec( ingestionSchema.getDataSchema(), new ParallelIndexIOConfig( - firehoseFactory, inputSource, ingestionSchema.getIOConfig().getInputFormat(), ingestionSchema.getIOConfig().isAppendToExisting(), diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/common/task/batch/parallel/LegacySinglePhaseSubTask.java b/indexing-service/src/main/java/org/apache/druid/indexing/common/task/batch/parallel/LegacySinglePhaseSubTask.java index 1daa99994274..27a242885c36 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/common/task/batch/parallel/LegacySinglePhaseSubTask.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/common/task/batch/parallel/LegacySinglePhaseSubTask.java @@ -71,9 +71,6 @@ public String getType() @Override public Set getInputSourceResources() { - if (getIngestionSchema().getIOConfig().getFirehoseFactory() != null) { - throw getInputSecurityOnFirehoseUnsupportedError(); - } return getIngestionSchema().getIOConfig().getInputSource() != null ? getIngestionSchema().getIOConfig().getInputSource().getTypes() .stream() diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/common/task/batch/parallel/ParallelIndexIOConfig.java b/indexing-service/src/main/java/org/apache/druid/indexing/common/task/batch/parallel/ParallelIndexIOConfig.java index c86446d91303..ff10e556de8f 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/common/task/batch/parallel/ParallelIndexIOConfig.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/common/task/batch/parallel/ParallelIndexIOConfig.java @@ -22,7 +22,6 @@ import com.fasterxml.jackson.annotation.JsonCreator; import com.fasterxml.jackson.annotation.JsonProperty; import com.fasterxml.jackson.annotation.JsonTypeName; -import org.apache.druid.data.input.FirehoseFactory; import org.apache.druid.data.input.InputFormat; import org.apache.druid.data.input.InputSource; import org.apache.druid.indexing.common.task.IndexTask.IndexIOConfig; @@ -37,26 +36,12 @@ public class ParallelIndexIOConfig extends IndexIOConfig { @JsonCreator public ParallelIndexIOConfig( - @JsonProperty("firehose") @Nullable FirehoseFactory firehoseFactory, @JsonProperty("inputSource") @Nullable InputSource inputSource, @JsonProperty("inputFormat") @Nullable InputFormat inputFormat, @JsonProperty("appendToExisting") @Nullable Boolean appendToExisting, @JsonProperty("dropExisting") @Nullable Boolean dropExisting ) { - super(firehoseFactory, inputSource, inputFormat, appendToExisting, dropExisting); - } - - // old constructor for backward compatibility - @Deprecated - public ParallelIndexIOConfig(FirehoseFactory firehoseFactory, @Nullable Boolean appendToExisting) - { - this(firehoseFactory, null, null, appendToExisting, null); - } - - @Deprecated - public ParallelIndexIOConfig(FirehoseFactory firehoseFactory, @Nullable Boolean appendToExisting, boolean dropExisting) - { - this(firehoseFactory, null, null, appendToExisting, dropExisting); + super(inputSource, inputFormat, appendToExisting, dropExisting); } } diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/common/task/batch/parallel/ParallelIndexSupervisorTask.java b/indexing-service/src/main/java/org/apache/druid/indexing/common/task/batch/parallel/ParallelIndexSupervisorTask.java index 4ca0f1ff80d7..6039a62bc984 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/common/task/batch/parallel/ParallelIndexSupervisorTask.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/common/task/batch/parallel/ParallelIndexSupervisorTask.java @@ -75,10 +75,10 @@ import org.apache.druid.segment.indexing.granularity.ArbitraryGranularitySpec; import org.apache.druid.segment.indexing.granularity.GranularitySpec; import org.apache.druid.segment.metadata.CentralizedDatasourceSchemaConfig; +import org.apache.druid.segment.realtime.ChatHandler; +import org.apache.druid.segment.realtime.ChatHandlers; import org.apache.druid.segment.realtime.appenderator.SegmentIdWithShardSpec; import org.apache.druid.segment.realtime.appenderator.TransactionalSegmentPublisher; -import org.apache.druid.segment.realtime.firehose.ChatHandler; -import org.apache.druid.segment.realtime.firehose.ChatHandlers; import org.apache.druid.server.security.Action; import org.apache.druid.server.security.AuthorizerMapper; import org.apache.druid.server.security.Resource; @@ -290,9 +290,6 @@ public String getType() @Override public Set getInputSourceResources() { - if (getIngestionSchema().getIOConfig().getFirehoseFactory() != null) { - throw getInputSecurityOnFirehoseUnsupportedError(); - } return getIngestionSchema().getIOConfig().getInputSource() != null ? getIngestionSchema().getIOConfig().getInputSource().getTypes() .stream() @@ -556,7 +553,7 @@ public TaskStatus runTask(TaskToolbox toolbox) throws Exception } else { if (!baseInputSource.isSplittable()) { LOG.warn( - "firehoseFactory[%s] is not splittable. Running sequentially.", + "inputSource[%s] is not splittable. Running sequentially.", baseInputSource.getClass().getSimpleName() ); } else if (ingestionSchema.getTuningConfig().getMaxNumConcurrentSubTasks() <= 1) { diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/common/task/batch/parallel/PartialDimensionCardinalityTask.java b/indexing-service/src/main/java/org/apache/druid/indexing/common/task/batch/parallel/PartialDimensionCardinalityTask.java index 6724a2ebb90c..d75304f38c08 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/common/task/batch/parallel/PartialDimensionCardinalityTask.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/common/task/batch/parallel/PartialDimensionCardinalityTask.java @@ -139,9 +139,6 @@ public String getType() @Override public Set getInputSourceResources() { - if (getIngestionSchema().getIOConfig().getFirehoseFactory() != null) { - throw getInputSecurityOnFirehoseUnsupportedError(); - } return getIngestionSchema().getIOConfig().getInputSource() != null ? getIngestionSchema().getIOConfig().getInputSource().getTypes() .stream() diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/common/task/batch/parallel/PartialDimensionDistributionTask.java b/indexing-service/src/main/java/org/apache/druid/indexing/common/task/batch/parallel/PartialDimensionDistributionTask.java index cb361a204ea3..4bb395420b3f 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/common/task/batch/parallel/PartialDimensionDistributionTask.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/common/task/batch/parallel/PartialDimensionDistributionTask.java @@ -180,9 +180,6 @@ public String getType() @Override public Set getInputSourceResources() { - if (getIngestionSchema().getIOConfig().getFirehoseFactory() != null) { - throw getInputSecurityOnFirehoseUnsupportedError(); - } return getIngestionSchema().getIOConfig().getInputSource() != null ? getIngestionSchema().getIOConfig().getInputSource().getTypes() .stream() diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/common/task/batch/parallel/PartialHashSegmentGenerateTask.java b/indexing-service/src/main/java/org/apache/druid/indexing/common/task/batch/parallel/PartialHashSegmentGenerateTask.java index 383fc7afb2e0..4c224e396d18 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/common/task/batch/parallel/PartialHashSegmentGenerateTask.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/common/task/batch/parallel/PartialHashSegmentGenerateTask.java @@ -135,9 +135,6 @@ public String getType() @Override public Set getInputSourceResources() { - if (getIngestionSchema().getIOConfig().getFirehoseFactory() != null) { - throw getInputSecurityOnFirehoseUnsupportedError(); - } return getIngestionSchema().getIOConfig().getInputSource() != null ? getIngestionSchema().getIOConfig().getInputSource().getTypes() .stream() diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/common/task/batch/parallel/PartialRangeSegmentGenerateTask.java b/indexing-service/src/main/java/org/apache/druid/indexing/common/task/batch/parallel/PartialRangeSegmentGenerateTask.java index cf7ae15a9a59..933df9ee7785 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/common/task/batch/parallel/PartialRangeSegmentGenerateTask.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/common/task/batch/parallel/PartialRangeSegmentGenerateTask.java @@ -153,9 +153,6 @@ public String getType() @Override public Set getInputSourceResources() { - if (getIngestionSchema().getIOConfig().getFirehoseFactory() != null) { - throw getInputSecurityOnFirehoseUnsupportedError(); - } return getIngestionSchema().getIOConfig().getInputSource() != null ? getIngestionSchema().getIOConfig().getInputSource().getTypes() .stream() diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/common/task/batch/parallel/PartialSegmentGenerateTask.java b/indexing-service/src/main/java/org/apache/druid/indexing/common/task/batch/parallel/PartialSegmentGenerateTask.java index c53557c66558..620d593467ad 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/common/task/batch/parallel/PartialSegmentGenerateTask.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/common/task/batch/parallel/PartialSegmentGenerateTask.java @@ -37,8 +37,8 @@ import org.apache.druid.indexing.common.task.TaskResource; import org.apache.druid.indexing.common.task.Tasks; import org.apache.druid.indexing.common.task.batch.parallel.iterator.IndexTaskInputRowIteratorBuilder; -import org.apache.druid.indexing.firehose.WindowedSegmentId; import org.apache.druid.indexing.input.DruidInputSource; +import org.apache.druid.indexing.input.WindowedSegmentId; import org.apache.druid.indexing.worker.shuffle.ShuffleDataSegmentPusher; import org.apache.druid.java.util.common.Pair; import org.apache.druid.segment.SegmentSchemaMapping; diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/common/task/batch/parallel/SinglePhaseParallelIndexTaskRunner.java b/indexing-service/src/main/java/org/apache/druid/indexing/common/task/batch/parallel/SinglePhaseParallelIndexTaskRunner.java index ce6aee98af35..a2a29b3cdd3e 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/common/task/batch/parallel/SinglePhaseParallelIndexTaskRunner.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/common/task/batch/parallel/SinglePhaseParallelIndexTaskRunner.java @@ -21,7 +21,6 @@ import com.google.common.annotations.VisibleForTesting; import org.apache.commons.lang3.mutable.MutableObject; -import org.apache.druid.data.input.FirehoseFactory; import org.apache.druid.data.input.InputSource; import org.apache.druid.data.input.InputSplit; import org.apache.druid.data.input.impl.SplittableInputSource; @@ -174,9 +173,7 @@ int estimateTotalNumSubTasks() throws IOException @VisibleForTesting SubTaskSpec newTaskSpec(InputSplit split) { - final FirehoseFactory firehoseFactory; final InputSource inputSource; - firehoseFactory = null; inputSource = baseInputSource.withSplit(split); final Map subtaskContext = new HashMap<>(getContext()); @@ -187,7 +184,6 @@ SubTaskSpec newTaskSpec(InputSplit split) new ParallelIndexIngestionSpec( ingestionSchema.getDataSchema(), new ParallelIndexIOConfig( - firehoseFactory, inputSource, ingestionSchema.getIOConfig().getInputFormat(), ingestionSchema.getIOConfig().isAppendToExisting(), diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/common/task/batch/parallel/SinglePhaseSubTask.java b/indexing-service/src/main/java/org/apache/druid/indexing/common/task/batch/parallel/SinglePhaseSubTask.java index cd05e50ff1c8..8d49a7f3dbe7 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/common/task/batch/parallel/SinglePhaseSubTask.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/common/task/batch/parallel/SinglePhaseSubTask.java @@ -60,13 +60,13 @@ import org.apache.druid.segment.indexing.granularity.ArbitraryGranularitySpec; import org.apache.druid.segment.indexing.granularity.GranularitySpec; import org.apache.druid.segment.metadata.CentralizedDatasourceSchemaConfig; +import org.apache.druid.segment.realtime.ChatHandler; import org.apache.druid.segment.realtime.SegmentGenerationMetrics; import org.apache.druid.segment.realtime.appenderator.Appenderator; import org.apache.druid.segment.realtime.appenderator.AppenderatorDriverAddResult; import org.apache.druid.segment.realtime.appenderator.BaseAppenderatorDriver; import org.apache.druid.segment.realtime.appenderator.BatchAppenderatorDriver; import org.apache.druid.segment.realtime.appenderator.SegmentsAndCommitMetadata; -import org.apache.druid.segment.realtime.firehose.ChatHandler; import org.apache.druid.server.security.Action; import org.apache.druid.server.security.AuthorizerMapper; import org.apache.druid.server.security.Resource; @@ -198,9 +198,6 @@ public String getType() @Override public Set getInputSourceResources() { - if (getIngestionSchema().getIOConfig().getFirehoseFactory() != null) { - throw getInputSecurityOnFirehoseUnsupportedError(); - } return getIngestionSchema().getIOConfig().getInputSource() != null ? getIngestionSchema().getIOConfig().getInputSource().getTypes() .stream() diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/common/task/batch/parallel/iterator/IndexTaskInputRowIteratorBuilder.java b/indexing-service/src/main/java/org/apache/druid/indexing/common/task/batch/parallel/iterator/IndexTaskInputRowIteratorBuilder.java index 89afdbb64d05..3e3d653e3c32 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/common/task/batch/parallel/iterator/IndexTaskInputRowIteratorBuilder.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/common/task/batch/parallel/iterator/IndexTaskInputRowIteratorBuilder.java @@ -19,7 +19,6 @@ package org.apache.druid.indexing.common.task.batch.parallel.iterator; -import org.apache.druid.data.input.Firehose; import org.apache.druid.data.input.HandlingInputRowIterator; import org.apache.druid.data.input.InputRow; import org.apache.druid.java.util.common.parsers.CloseableIterator; @@ -34,7 +33,6 @@ public interface IndexTaskInputRowIteratorBuilder /** * @param granularitySpec {@link GranularitySpec} for the {@link org.apache.druid.segment.indexing.DataSchema} - * associated with the {@link Firehose}. */ IndexTaskInputRowIteratorBuilder granularitySpec(GranularitySpec granularitySpec); diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/input/DruidInputSource.java b/indexing-service/src/main/java/org/apache/druid/indexing/input/DruidInputSource.java index dd1998645b34..84c17f1a8fa0 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/input/DruidInputSource.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/input/DruidInputSource.java @@ -51,7 +51,6 @@ import org.apache.druid.indexing.common.TaskToolbox; import org.apache.druid.indexing.common.actions.RetrieveUsedSegmentsAction; import org.apache.druid.indexing.common.config.TaskConfig; -import org.apache.druid.indexing.firehose.WindowedSegmentId; import org.apache.druid.java.util.common.CloseableIterators; import org.apache.druid.java.util.common.IAE; import org.apache.druid.java.util.common.ISE; @@ -133,7 +132,7 @@ public class DruidInputSource extends AbstractInputSource implements SplittableI private final String dataSource; // Exactly one of interval and segmentIds should be non-null. Typically 'interval' is specified directly - // by the user creating this firehose and 'segmentIds' is used for sub-tasks if it is split for parallel + // by the user creating this input source and 'segmentIds' is used for sub-tasks if it is split for parallel // batch ingestion. @Nullable private final Interval interval; @@ -164,7 +163,7 @@ public class DruidInputSource extends AbstractInputSource implements SplittableI public DruidInputSource( @JsonProperty("dataSource") final String dataSource, @JsonProperty("interval") @Nullable Interval interval, - // Specifying "segments" is intended only for when this FirehoseFactory has split itself, + // Specifying "segments" is intended only for when this input source has split itself, // not for direct end user use. @JsonProperty("segments") @Nullable List segmentIds, @JsonProperty("filter") DimFilter dimFilter, diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/input/DruidSegmentReader.java b/indexing-service/src/main/java/org/apache/druid/indexing/input/DruidSegmentReader.java index d048cf4d8b91..42618556d003 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/input/DruidSegmentReader.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/input/DruidSegmentReader.java @@ -56,7 +56,7 @@ import org.apache.druid.segment.column.ColumnType; import org.apache.druid.segment.data.IndexedInts; import org.apache.druid.segment.filter.Filters; -import org.apache.druid.segment.realtime.firehose.WindowedStorageAdapter; +import org.apache.druid.segment.realtime.WindowedStorageAdapter; import org.apache.druid.utils.CloseableUtils; import org.apache.druid.utils.CollectionUtils; import org.joda.time.Interval; diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/firehose/WindowedSegmentId.java b/indexing-service/src/main/java/org/apache/druid/indexing/input/WindowedSegmentId.java similarity index 98% rename from indexing-service/src/main/java/org/apache/druid/indexing/firehose/WindowedSegmentId.java rename to indexing-service/src/main/java/org/apache/druid/indexing/input/WindowedSegmentId.java index b55510a7e219..8232aba04460 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/firehose/WindowedSegmentId.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/input/WindowedSegmentId.java @@ -17,7 +17,7 @@ * under the License. */ -package org.apache.druid.indexing.firehose; +package org.apache.druid.indexing.input; import com.fasterxml.jackson.annotation.JsonCreator; import com.fasterxml.jackson.annotation.JsonProperty; diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/overlord/sampler/InputSourceSampler.java b/indexing-service/src/main/java/org/apache/druid/indexing/overlord/sampler/InputSourceSampler.java index a84cb4e1979c..5018795cbb4f 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/overlord/sampler/InputSourceSampler.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/overlord/sampler/InputSourceSampler.java @@ -76,8 +76,8 @@ public class InputSourceSampler null ); - // We want to be able to sort the list of processed results back into the same order that we read them from the - // firehose so that the rows in the data loader are not always changing. To do this, we add a temporary column to the + // We want to be able to sort the list of processed results back into the same order that we read them from the input + // source so that the rows in the data loader are not always changing. To do this, we add a temporary column to the // InputRow (in SamplerInputRow) and tag each row with a sortKey. We use an aggregator so that it will not affect // rollup, and we use a longMin aggregator so that as rows get rolled up, the earlier rows stay stable and later // rows may get rolled into these rows. After getting the results back from the IncrementalIndex, we sort by this diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/seekablestream/SeekableStreamIndexTask.java b/indexing-service/src/main/java/org/apache/druid/indexing/seekablestream/SeekableStreamIndexTask.java index ee4eed572dff..41cd084cd960 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/seekablestream/SeekableStreamIndexTask.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/seekablestream/SeekableStreamIndexTask.java @@ -49,10 +49,10 @@ import org.apache.druid.segment.incremental.ParseExceptionHandler; import org.apache.druid.segment.incremental.RowIngestionMeters; import org.apache.druid.segment.indexing.DataSchema; +import org.apache.druid.segment.realtime.ChatHandler; import org.apache.druid.segment.realtime.SegmentGenerationMetrics; import org.apache.druid.segment.realtime.appenderator.Appenderator; import org.apache.druid.segment.realtime.appenderator.StreamAppenderatorDriver; -import org.apache.druid.segment.realtime.firehose.ChatHandler; import org.apache.druid.server.security.AuthorizerMapper; import org.apache.druid.timeline.partition.NumberedPartialShardSpec; import org.checkerframework.checker.nullness.qual.MonotonicNonNull; diff --git a/indexing-service/src/main/java/org/apache/druid/indexing/seekablestream/SeekableStreamIndexTaskRunner.java b/indexing-service/src/main/java/org/apache/druid/indexing/seekablestream/SeekableStreamIndexTaskRunner.java index fad7b923ca97..d347fd815038 100644 --- a/indexing-service/src/main/java/org/apache/druid/indexing/seekablestream/SeekableStreamIndexTaskRunner.java +++ b/indexing-service/src/main/java/org/apache/druid/indexing/seekablestream/SeekableStreamIndexTaskRunner.java @@ -81,13 +81,13 @@ import org.apache.druid.segment.incremental.ParseExceptionHandler; import org.apache.druid.segment.incremental.ParseExceptionReport; import org.apache.druid.segment.incremental.RowIngestionMeters; +import org.apache.druid.segment.realtime.ChatHandler; import org.apache.druid.segment.realtime.SegmentGenerationMetrics; import org.apache.druid.segment.realtime.appenderator.Appenderator; import org.apache.druid.segment.realtime.appenderator.AppenderatorDriverAddResult; import org.apache.druid.segment.realtime.appenderator.SegmentsAndCommitMetadata; import org.apache.druid.segment.realtime.appenderator.StreamAppenderator; import org.apache.druid.segment.realtime.appenderator.StreamAppenderatorDriver; -import org.apache.druid.segment.realtime.firehose.ChatHandler; import org.apache.druid.server.security.Access; import org.apache.druid.server.security.Action; import org.apache.druid.server.security.AuthorizerMapper; diff --git a/indexing-service/src/test/java/org/apache/druid/indexing/common/TaskToolboxTest.java b/indexing-service/src/test/java/org/apache/druid/indexing/common/TaskToolboxTest.java index e1ac9482436b..a4f297c8e335 100644 --- a/indexing-service/src/test/java/org/apache/druid/indexing/common/TaskToolboxTest.java +++ b/indexing-service/src/test/java/org/apache/druid/indexing/common/TaskToolboxTest.java @@ -51,9 +51,9 @@ import org.apache.druid.segment.loading.SegmentLoaderConfig; import org.apache.druid.segment.loading.SegmentLocalCacheManager; import org.apache.druid.segment.metadata.CentralizedDatasourceSchemaConfig; +import org.apache.druid.segment.realtime.NoopChatHandlerProvider; import org.apache.druid.segment.realtime.appenderator.AppenderatorsManager; import org.apache.druid.segment.realtime.appenderator.UnifiedIndexerAppenderatorsManager; -import org.apache.druid.segment.realtime.firehose.NoopChatHandlerProvider; import org.apache.druid.server.DruidNode; import org.apache.druid.server.coordination.DataSegmentAnnouncer; import org.apache.druid.server.coordination.DataSegmentServerAnnouncer; diff --git a/indexing-service/src/test/java/org/apache/druid/indexing/common/TestFirehose.java b/indexing-service/src/test/java/org/apache/druid/indexing/common/TestFirehose.java deleted file mode 100644 index 1572373151bf..000000000000 --- a/indexing-service/src/test/java/org/apache/druid/indexing/common/TestFirehose.java +++ /dev/null @@ -1,118 +0,0 @@ -/* - * Licensed to the Apache Software Foundation (ASF) under one - * or more contributor license agreements. See the NOTICE file - * distributed with this work for additional information - * regarding copyright ownership. The ASF licenses this file - * to you under the Apache License, Version 2.0 (the - * "License"); you may not use this file except in compliance - * with the License. You may obtain a copy of the License at - * - * http://www.apache.org/licenses/LICENSE-2.0 - * - * Unless required by applicable law or agreed to in writing, - * software distributed under the License is distributed on an - * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY - * KIND, either express or implied. See the License for the - * specific language governing permissions and limitations - * under the License. - */ - -package org.apache.druid.indexing.common; - -import org.apache.druid.data.input.Firehose; -import org.apache.druid.data.input.FirehoseFactory; -import org.apache.druid.data.input.InputRow; -import org.apache.druid.data.input.impl.InputRowParser; -import org.apache.druid.data.input.impl.StringInputRowParser; -import org.apache.druid.java.util.common.parsers.ParseException; - -import java.io.File; -import java.util.ArrayDeque; -import java.util.Collections; -import java.util.Deque; -import java.util.List; -import java.util.Optional; - -public class TestFirehose implements Firehose -{ - public static class TestFirehoseFactory implements FirehoseFactory - { - private boolean waitForClose = true; - - @Override - @SuppressWarnings("unchecked") - public Firehose connect(InputRowParser parser, File temporaryDirectory) throws ParseException - { - return new TestFirehose(parser, waitForClose, Collections.emptyList()); - } - } - - public static final String FAIL_DIM = "__fail__"; - - private final Deque> queue = new ArrayDeque<>(); - - private InputRowParser parser; - private boolean closed; - - private TestFirehose(InputRowParser parser, boolean waitForClose, List seedRows) - { - this.parser = parser; - this.closed = !waitForClose; - - if (parser instanceof StringInputRowParser) { - ((StringInputRowParser) parser).startFileFromBeginning(); - } - - if (seedRows != null) { - seedRows.stream().map(Optional::ofNullable).forEach(queue::add); - } - } - - public void addRows(List rows) - { - synchronized (this) { - rows.stream().map(Optional::ofNullable).forEach(queue::add); - notifyAll(); - } - } - - @Override - public boolean hasMore() - { - try { - synchronized (this) { - while (queue.isEmpty() && !closed) { - wait(); - } - return !queue.isEmpty(); - } - } - catch (InterruptedException e) { - Thread.currentThread().interrupt(); - throw new RuntimeException(e); - } - } - - @Override - public InputRow nextRow() - { - synchronized (this) { - final InputRow row = parser instanceof StringInputRowParser - ? ((StringInputRowParser) parser).parse((String) queue.removeFirst().orElse(null)) - : (InputRow) parser.parseBatch(queue.removeFirst().orElse(null)).get(0); - if (row != null && row.getRaw(FAIL_DIM) != null) { - throw new ParseException(null, FAIL_DIM); - } - return row; - } - } - - @Override - public void close() - { - synchronized (this) { - closed = true; - notifyAll(); - } - } -} diff --git a/indexing-service/src/test/java/org/apache/druid/indexing/common/TestIndexTask.java b/indexing-service/src/test/java/org/apache/druid/indexing/common/TestIndexTask.java index c69ac562821b..f6732f68a6c4 100644 --- a/indexing-service/src/test/java/org/apache/druid/indexing/common/TestIndexTask.java +++ b/indexing-service/src/test/java/org/apache/druid/indexing/common/TestIndexTask.java @@ -64,7 +64,6 @@ public TestIndexTask( new IndexIngestionSpec( new DataSchema(dataSource, null, new AggregatorFactory[]{}, null, null, mapper), new IndexTask.IndexIOConfig( - null, new LocalInputSource(new File("lol"), "rofl"), new JsonInputFormat(null, null, null, null, null), false, diff --git a/indexing-service/src/test/java/org/apache/druid/indexing/common/TestUtils.java b/indexing-service/src/test/java/org/apache/druid/indexing/common/TestUtils.java index 7dda3b8ff62c..ae5ba44f7f09 100644 --- a/indexing-service/src/test/java/org/apache/druid/indexing/common/TestUtils.java +++ b/indexing-service/src/test/java/org/apache/druid/indexing/common/TestUtils.java @@ -20,7 +20,6 @@ package org.apache.druid.indexing.common; import com.fasterxml.jackson.databind.InjectableValues; -import com.fasterxml.jackson.databind.Module; import com.fasterxml.jackson.databind.ObjectMapper; import com.fasterxml.jackson.databind.jsontype.NamedType; import com.fasterxml.jackson.databind.module.SimpleModule; @@ -30,7 +29,6 @@ import org.apache.druid.data.input.impl.NoopInputFormat; import org.apache.druid.data.input.impl.NoopInputSource; import org.apache.druid.guice.DruidSecondaryModule; -import org.apache.druid.guice.FirehoseModule; import org.apache.druid.indexing.common.stats.DropwizardRowIngestionMetersFactory; import org.apache.druid.indexing.common.task.TestAppenderatorsManager; import org.apache.druid.indexing.common.task.batch.parallel.ParallelIndexSupervisorTaskClientProvider; @@ -49,15 +47,14 @@ import org.apache.druid.segment.incremental.RowIngestionMetersFactory; import org.apache.druid.segment.loading.LocalDataSegmentPuller; import org.apache.druid.segment.loading.LocalLoadSpec; +import org.apache.druid.segment.realtime.ChatHandlerProvider; +import org.apache.druid.segment.realtime.NoopChatHandlerProvider; import org.apache.druid.segment.realtime.appenderator.AppenderatorsManager; -import org.apache.druid.segment.realtime.firehose.ChatHandlerProvider; -import org.apache.druid.segment.realtime.firehose.NoopChatHandlerProvider; import org.apache.druid.segment.writeout.OffHeapMemorySegmentWriteOutMediumFactory; import org.apache.druid.server.security.AuthConfig; import org.apache.druid.server.security.AuthorizerMapper; import org.apache.druid.timeline.DataSegment.PruneSpecsHolder; -import java.util.List; import java.util.concurrent.TimeUnit; /** @@ -122,9 +119,6 @@ public void setupModule(SetupContext context) } ); DruidSecondaryModule.setupAnnotationIntrospector(jsonMapper, TestHelper.makeAnnotationIntrospector()); - - List firehoseModules = new FirehoseModule().getJacksonModules(); - firehoseModules.forEach(jsonMapper::registerModule); } public ObjectMapper getTestObjectMapper() diff --git a/indexing-service/src/test/java/org/apache/druid/indexing/common/task/BatchAppenderatorsTest.java b/indexing-service/src/test/java/org/apache/druid/indexing/common/task/BatchAppenderatorsTest.java index f789c1e641f7..278a27f24bba 100644 --- a/indexing-service/src/test/java/org/apache/druid/indexing/common/task/BatchAppenderatorsTest.java +++ b/indexing-service/src/test/java/org/apache/druid/indexing/common/task/BatchAppenderatorsTest.java @@ -52,12 +52,12 @@ import org.apache.druid.segment.join.NoopJoinableFactory; import org.apache.druid.segment.loading.DataSegmentPusher; import org.apache.druid.segment.metadata.CentralizedDatasourceSchemaConfig; +import org.apache.druid.segment.realtime.NoopChatHandlerProvider; import org.apache.druid.segment.realtime.SegmentGenerationMetrics; import org.apache.druid.segment.realtime.appenderator.Appenderator; import org.apache.druid.segment.realtime.appenderator.AppenderatorConfig; import org.apache.druid.segment.realtime.appenderator.AppenderatorImpl; import org.apache.druid.segment.realtime.appenderator.BatchAppenderator; -import org.apache.druid.segment.realtime.firehose.NoopChatHandlerProvider; import org.apache.druid.segment.writeout.OffHeapMemorySegmentWriteOutMediumFactory; import org.apache.druid.segment.writeout.SegmentWriteOutMediumFactory; import org.apache.druid.server.security.AuthTestUtils; diff --git a/indexing-service/src/test/java/org/apache/druid/indexing/common/task/ClientCompactionTaskQuerySerdeTest.java b/indexing-service/src/test/java/org/apache/druid/indexing/common/task/ClientCompactionTaskQuerySerdeTest.java index cd942a9a7668..55898357d663 100644 --- a/indexing-service/src/test/java/org/apache/druid/indexing/common/task/ClientCompactionTaskQuerySerdeTest.java +++ b/indexing-service/src/test/java/org/apache/druid/indexing/common/task/ClientCompactionTaskQuerySerdeTest.java @@ -60,9 +60,9 @@ import org.apache.druid.segment.data.CompressionStrategy; import org.apache.druid.segment.incremental.OnheapIncrementalIndex; import org.apache.druid.segment.incremental.RowIngestionMetersFactory; +import org.apache.druid.segment.realtime.ChatHandlerProvider; +import org.apache.druid.segment.realtime.NoopChatHandlerProvider; import org.apache.druid.segment.realtime.appenderator.AppenderatorsManager; -import org.apache.druid.segment.realtime.firehose.ChatHandlerProvider; -import org.apache.druid.segment.realtime.firehose.NoopChatHandlerProvider; import org.apache.druid.segment.writeout.TmpFileSegmentWriteOutMediumFactory; import org.apache.druid.server.lookup.cache.LookupLoadingSpec; import org.apache.druid.server.security.AuthTestUtils; diff --git a/indexing-service/src/test/java/org/apache/druid/indexing/common/task/CompactionTaskParallelRunTest.java b/indexing-service/src/test/java/org/apache/druid/indexing/common/task/CompactionTaskParallelRunTest.java index 188ea3cdd071..1b742971eb95 100644 --- a/indexing-service/src/test/java/org/apache/druid/indexing/common/task/CompactionTaskParallelRunTest.java +++ b/indexing-service/src/test/java/org/apache/druid/indexing/common/task/CompactionTaskParallelRunTest.java @@ -49,8 +49,8 @@ import org.apache.druid.indexing.common.task.batch.parallel.ParallelIndexIngestionSpec; import org.apache.druid.indexing.common.task.batch.parallel.ParallelIndexSupervisorTask; import org.apache.druid.indexing.common.task.batch.parallel.ParallelIndexTuningConfig; -import org.apache.druid.indexing.firehose.WindowedSegmentId; import org.apache.druid.indexing.input.DruidInputSource; +import org.apache.druid.indexing.input.WindowedSegmentId; import org.apache.druid.java.util.common.Intervals; import org.apache.druid.java.util.common.granularity.Granularities; import org.apache.druid.query.aggregation.AggregatorFactory; @@ -922,7 +922,6 @@ protected TaskToolbox createTaskToolbox(Task task, TaskActionClient actionClient private void runIndexTask(@Nullable PartitionsSpec partitionsSpec, boolean appendToExisting) { ParallelIndexIOConfig ioConfig = new ParallelIndexIOConfig( - null, new LocalInputSource(inputDir, "druid*"), new CsvInputFormat( Arrays.asList("ts", "dim", "val"), diff --git a/indexing-service/src/test/java/org/apache/druid/indexing/common/task/CompactionTaskRunTest.java b/indexing-service/src/test/java/org/apache/druid/indexing/common/task/CompactionTaskRunTest.java index 0c274a019668..c00a926442ce 100644 --- a/indexing-service/src/test/java/org/apache/druid/indexing/common/task/CompactionTaskRunTest.java +++ b/indexing-service/src/test/java/org/apache/druid/indexing/common/task/CompactionTaskRunTest.java @@ -94,8 +94,8 @@ import org.apache.druid.segment.loading.StorageLocationConfig; import org.apache.druid.segment.loading.TombstoneLoadSpec; import org.apache.druid.segment.metadata.CentralizedDatasourceSchemaConfig; -import org.apache.druid.segment.realtime.firehose.NoopChatHandlerProvider; -import org.apache.druid.segment.realtime.firehose.WindowedStorageAdapter; +import org.apache.druid.segment.realtime.NoopChatHandlerProvider; +import org.apache.druid.segment.realtime.WindowedStorageAdapter; import org.apache.druid.server.security.AuthTestUtils; import org.apache.druid.timeline.CompactionState; import org.apache.druid.timeline.DataSegment; diff --git a/indexing-service/src/test/java/org/apache/druid/indexing/common/task/CompactionTaskTest.java b/indexing-service/src/test/java/org/apache/druid/indexing/common/task/CompactionTaskTest.java index 25dc84d98b79..562347a7f972 100644 --- a/indexing-service/src/test/java/org/apache/druid/indexing/common/task/CompactionTaskTest.java +++ b/indexing-service/src/test/java/org/apache/druid/indexing/common/task/CompactionTaskTest.java @@ -123,9 +123,9 @@ import org.apache.druid.segment.loading.NoopSegmentCacheManager; import org.apache.druid.segment.loading.SegmentCacheManager; import org.apache.druid.segment.metadata.CentralizedDatasourceSchemaConfig; +import org.apache.druid.segment.realtime.ChatHandlerProvider; +import org.apache.druid.segment.realtime.NoopChatHandlerProvider; import org.apache.druid.segment.realtime.appenderator.AppenderatorsManager; -import org.apache.druid.segment.realtime.firehose.ChatHandlerProvider; -import org.apache.druid.segment.realtime.firehose.NoopChatHandlerProvider; import org.apache.druid.segment.selector.settable.SettableColumnValueSelector; import org.apache.druid.segment.writeout.OffHeapMemorySegmentWriteOutMediumFactory; import org.apache.druid.server.lookup.cache.LookupLoadingSpec; diff --git a/indexing-service/src/test/java/org/apache/druid/indexing/common/task/IndexIngestionSpecTest.java b/indexing-service/src/test/java/org/apache/druid/indexing/common/task/IndexIngestionSpecTest.java index 7555c6b1e378..ab953ba954ac 100644 --- a/indexing-service/src/test/java/org/apache/druid/indexing/common/task/IndexIngestionSpecTest.java +++ b/indexing-service/src/test/java/org/apache/druid/indexing/common/task/IndexIngestionSpecTest.java @@ -54,7 +54,6 @@ public void testParserAndInputFormat() null ), new IndexIOConfig( - null, new NoopInputSource(), new NoopInputFormat(), null, @@ -79,7 +78,6 @@ public void testParserAndInputSource() null ), new IndexIOConfig( - null, new NoopInputSource(), null, null, diff --git a/indexing-service/src/test/java/org/apache/druid/indexing/common/task/IndexTaskTest.java b/indexing-service/src/test/java/org/apache/druid/indexing/common/task/IndexTaskTest.java index 40be38445857..3ec5b8d0aa8a 100644 --- a/indexing-service/src/test/java/org/apache/druid/indexing/common/task/IndexTaskTest.java +++ b/indexing-service/src/test/java/org/apache/druid/indexing/common/task/IndexTaskTest.java @@ -93,7 +93,7 @@ import org.apache.druid.segment.loading.SegmentLocalCacheManager; import org.apache.druid.segment.loading.StorageLocation; import org.apache.druid.segment.loading.StorageLocationConfig; -import org.apache.druid.segment.realtime.firehose.WindowedStorageAdapter; +import org.apache.druid.segment.realtime.WindowedStorageAdapter; import org.apache.druid.segment.transform.ExpressionTransform; import org.apache.druid.segment.transform.TransformSpec; import org.apache.druid.server.metrics.NoopServiceEmitter; @@ -244,7 +244,6 @@ public void testCorrectInputSourceResources() null ), new IndexIOConfig( - null, new LocalInputSource(tmpDir, "druid*"), DEFAULT_INPUT_FORMAT, false, @@ -295,7 +294,6 @@ public void testIngestNullOnlyColumns() throws Exception null ), new IndexIOConfig( - null, new LocalInputSource(tmpDir, "druid*"), DEFAULT_INPUT_FORMAT, false, @@ -358,7 +356,6 @@ public void testIngestNullOnlyColumns_storeEmptyColumnsOff_shouldNotStoreEmptyCo null ), new IndexIOConfig( - null, new LocalInputSource(tmpDir, "druid*"), DEFAULT_INPUT_FORMAT, false, @@ -2727,7 +2724,6 @@ private static IndexIngestionSpec createIngestionSpec( transformSpec ), new IndexIOConfig( - null, new LocalInputSource(baseDir, "druid*"), inputFormat, appendToExisting, @@ -2755,7 +2751,6 @@ private static IndexIngestionSpec createIngestionSpec( objectMapper ), new IndexIOConfig( - null, new LocalInputSource(baseDir, "druid*"), createInputFormatFromParseSpec(parseSpec), appendToExisting, diff --git a/indexing-service/src/test/java/org/apache/druid/indexing/common/task/IngestionTestBase.java b/indexing-service/src/test/java/org/apache/druid/indexing/common/task/IngestionTestBase.java index ca4d7ee7fcad..f3b2729b8876 100644 --- a/indexing-service/src/test/java/org/apache/druid/indexing/common/task/IngestionTestBase.java +++ b/indexing-service/src/test/java/org/apache/druid/indexing/common/task/IngestionTestBase.java @@ -23,13 +23,11 @@ import com.google.common.base.Optional; import com.google.common.util.concurrent.Futures; import com.google.common.util.concurrent.ListenableFuture; -import org.apache.druid.data.input.FirehoseFactory; import org.apache.druid.data.input.InputFormat; import org.apache.druid.data.input.impl.CSVParseSpec; import org.apache.druid.data.input.impl.CsvInputFormat; import org.apache.druid.data.input.impl.DelimitedInputFormat; import org.apache.druid.data.input.impl.DelimitedParseSpec; -import org.apache.druid.data.input.impl.InputRowParser; import org.apache.druid.data.input.impl.JSONParseSpec; import org.apache.druid.data.input.impl.JsonInputFormat; import org.apache.druid.data.input.impl.ParseSpec; @@ -85,7 +83,7 @@ import org.apache.druid.segment.metadata.CentralizedDatasourceSchemaConfig; import org.apache.druid.segment.metadata.SegmentSchemaCache; import org.apache.druid.segment.metadata.SegmentSchemaManager; -import org.apache.druid.segment.realtime.firehose.NoopChatHandlerProvider; +import org.apache.druid.segment.realtime.NoopChatHandlerProvider; import org.apache.druid.server.DruidNode; import org.apache.druid.server.metrics.NoopServiceEmitter; import org.apache.druid.server.security.AuthTestUtils; @@ -300,8 +298,7 @@ public IndexMergerV9Factory getIndexMergerV9Factory() } /** - * Converts ParseSpec to InputFormat for indexing tests. To be used until {@link FirehoseFactory} - * & {@link InputRowParser} is deprecated and removed. + * Converts ParseSpec to InputFormat for indexing tests. Used for backwards compatibility */ public static InputFormat createInputFormatFromParseSpec(ParseSpec parseSpec) { diff --git a/indexing-service/src/test/java/org/apache/druid/indexing/common/task/TaskSerdeTest.java b/indexing-service/src/test/java/org/apache/druid/indexing/common/task/TaskSerdeTest.java index 9cf6c0102fbe..e6ea0e1329ae 100644 --- a/indexing-service/src/test/java/org/apache/druid/indexing/common/task/TaskSerdeTest.java +++ b/indexing-service/src/test/java/org/apache/druid/indexing/common/task/TaskSerdeTest.java @@ -19,19 +19,14 @@ package org.apache.druid.indexing.common.task; -import com.fasterxml.jackson.databind.Module; import com.fasterxml.jackson.databind.ObjectMapper; import com.fasterxml.jackson.databind.jsontype.NamedType; import com.google.common.collect.ImmutableList; import com.google.common.collect.ImmutableMap; -import org.apache.druid.data.input.Firehose; -import org.apache.druid.data.input.FirehoseFactory; import org.apache.druid.data.input.impl.DimensionsSpec; -import org.apache.druid.data.input.impl.InputRowParser; import org.apache.druid.data.input.impl.LocalInputSource; import org.apache.druid.data.input.impl.NoopInputFormat; import org.apache.druid.data.input.impl.TimestampSpec; -import org.apache.druid.guice.FirehoseModule; import org.apache.druid.indexer.HadoopIOConfig; import org.apache.druid.indexer.HadoopIngestionSpec; import org.apache.druid.indexer.partitions.DynamicPartitionsSpec; @@ -70,14 +65,9 @@ public TaskSerdeTest() { TestUtils testUtils = new TestUtils(); jsonMapper = testUtils.getTestObjectMapper(); - - for (final Module jacksonModule : new FirehoseModule().getJacksonModules()) { - jsonMapper.registerModule(jacksonModule); - } jsonMapper.registerSubtypes( new NamedType(ParallelIndexTuningConfig.class, "index_parallel"), - new NamedType(IndexTuningConfig.class, "index"), - new NamedType(MockFirehoseFactory.class, "mock") + new NamedType(IndexTuningConfig.class, "index") ); } @@ -242,7 +232,7 @@ public void testIndexTaskSerde() throws Exception ), null ), - new IndexIOConfig(null, new LocalInputSource(new File("lol"), "rofl"), new NoopInputFormat(), true, false), + new IndexIOConfig(new LocalInputSource(new File("lol"), "rofl"), new NoopInputFormat(), true, false), TuningConfigBuilder.forIndexTask() .withMaxRowsInMemory(10) .withPartitionsSpec(new DynamicPartitionsSpec(10000, null)) @@ -310,7 +300,7 @@ public void testIndexTaskwithResourceSerde() throws Exception ), null ), - new IndexIOConfig(null, new LocalInputSource(new File("lol"), "rofl"), new NoopInputFormat(), true, false), + new IndexIOConfig(new LocalInputSource(new File("lol"), "rofl"), new NoopInputFormat(), true, false), TuningConfigBuilder.forIndexTask() .withMaxRowsInMemory(10) .withForceGuaranteedRollup(false) @@ -322,10 +312,6 @@ public void testIndexTaskwithResourceSerde() throws Exception null ); - for (final Module jacksonModule : new FirehoseModule().getJacksonModules()) { - jsonMapper.registerModule(jacksonModule); - } - final String json = jsonMapper.writeValueAsString(task); Thread.sleep(100); // Just want to run the clock a bit to make sure the task id doesn't change @@ -462,15 +448,6 @@ public void testHadoopIndexTaskSerde() throws Exception Assert.assertEquals("blah", task2.getClasspathPrefix()); } - private static class MockFirehoseFactory implements FirehoseFactory - { - @Override - public Firehose connect(InputRowParser parser, File temporaryDirectory) - { - return null; - } - } - @Test public void testHadoopIndexTaskWithContextSerde() throws Exception { diff --git a/indexing-service/src/test/java/org/apache/druid/indexing/common/task/batch/parallel/AbstractMultiPhaseParallelIndexingTest.java b/indexing-service/src/test/java/org/apache/druid/indexing/common/task/batch/parallel/AbstractMultiPhaseParallelIndexingTest.java index b11ed53b8048..44b8284f407d 100644 --- a/indexing-service/src/test/java/org/apache/druid/indexing/common/task/batch/parallel/AbstractMultiPhaseParallelIndexingTest.java +++ b/indexing-service/src/test/java/org/apache/druid/indexing/common/task/batch/parallel/AbstractMultiPhaseParallelIndexingTest.java @@ -215,7 +215,6 @@ protected ParallelIndexSupervisorTask createTask( if (useInputFormatApi) { Preconditions.checkArgument(parseSpec == null); ParallelIndexIOConfig ioConfig = new ParallelIndexIOConfig( - null, new LocalInputSource(inputDirectory, filter), inputFormat, appendToExisting, @@ -236,7 +235,6 @@ protected ParallelIndexSupervisorTask createTask( } else { Preconditions.checkArgument(inputFormat == null && parseSpec != null); ParallelIndexIOConfig ioConfig = new ParallelIndexIOConfig( - null, new LocalInputSource(inputDirectory, filter), createInputFormatFromParseSpec(parseSpec), appendToExisting, diff --git a/indexing-service/src/test/java/org/apache/druid/indexing/common/task/batch/parallel/AbstractParallelIndexSupervisorTaskTest.java b/indexing-service/src/test/java/org/apache/druid/indexing/common/task/batch/parallel/AbstractParallelIndexSupervisorTaskTest.java index ea6f9d963fef..19feeaf44c45 100644 --- a/indexing-service/src/test/java/org/apache/druid/indexing/common/task/batch/parallel/AbstractParallelIndexSupervisorTaskTest.java +++ b/indexing-service/src/test/java/org/apache/druid/indexing/common/task/batch/parallel/AbstractParallelIndexSupervisorTaskTest.java @@ -96,10 +96,10 @@ import org.apache.druid.segment.loading.NoopDataSegmentKiller; import org.apache.druid.segment.loading.StorageLocationConfig; import org.apache.druid.segment.metadata.CentralizedDatasourceSchemaConfig; +import org.apache.druid.segment.realtime.ChatHandlerProvider; +import org.apache.druid.segment.realtime.NoopChatHandlerProvider; import org.apache.druid.segment.realtime.appenderator.AppenderatorsManager; import org.apache.druid.segment.realtime.appenderator.SegmentIdWithShardSpec; -import org.apache.druid.segment.realtime.firehose.ChatHandlerProvider; -import org.apache.druid.segment.realtime.firehose.NoopChatHandlerProvider; import org.apache.druid.server.DruidNode; import org.apache.druid.server.security.AuthConfig; import org.apache.druid.server.security.AuthTestUtils; diff --git a/indexing-service/src/test/java/org/apache/druid/indexing/common/task/batch/parallel/HashPartitionTaskKillTest.java b/indexing-service/src/test/java/org/apache/druid/indexing/common/task/batch/parallel/HashPartitionTaskKillTest.java index e71dc6db36d3..b8c59d042a31 100644 --- a/indexing-service/src/test/java/org/apache/druid/indexing/common/task/batch/parallel/HashPartitionTaskKillTest.java +++ b/indexing-service/src/test/java/org/apache/druid/indexing/common/task/batch/parallel/HashPartitionTaskKillTest.java @@ -244,7 +244,6 @@ private ParallelIndexSupervisorTask createTestTask( if (useInputFormatApi) { Preconditions.checkArgument(parseSpec == null); ParallelIndexIOConfig ioConfig = new ParallelIndexIOConfig( - null, new LocalInputSource(inputDir, filter), inputFormat, appendToExisting, @@ -265,7 +264,6 @@ private ParallelIndexSupervisorTask createTestTask( } else { Preconditions.checkArgument(inputFormat == null); ParallelIndexIOConfig ioConfig = new ParallelIndexIOConfig( - null, new LocalInputSource(inputDir, filter), createInputFormatFromParseSpec(parseSpec), appendToExisting, diff --git a/indexing-service/src/test/java/org/apache/druid/indexing/common/task/batch/parallel/MultiPhaseParallelIndexingWithNullColumnTest.java b/indexing-service/src/test/java/org/apache/druid/indexing/common/task/batch/parallel/MultiPhaseParallelIndexingWithNullColumnTest.java index 1448a5723f36..0d19cd86e03c 100644 --- a/indexing-service/src/test/java/org/apache/druid/indexing/common/task/batch/parallel/MultiPhaseParallelIndexingWithNullColumnTest.java +++ b/indexing-service/src/test/java/org/apache/druid/indexing/common/task/batch/parallel/MultiPhaseParallelIndexingWithNullColumnTest.java @@ -140,7 +140,6 @@ public void testIngestNullColumn() throws JsonProcessingException null ), new ParallelIndexIOConfig( - null, getInputSource(), JSON_FORMAT, false, @@ -191,7 +190,6 @@ public void testIngestNullColumn_useFieldDiscovery_includeAllDimensions_shouldSt null ), new ParallelIndexIOConfig( - null, getInputSource(), new JsonInputFormat( new JSONPathSpec(true, null), @@ -252,7 +250,6 @@ public void testIngestNullColumn_explicitPathSpec_useFieldDiscovery_includeAllDi null ), new ParallelIndexIOConfig( - null, getInputSource(), new JsonInputFormat( new JSONPathSpec( @@ -321,7 +318,6 @@ public void testIngestNullColumn_storeEmptyColumnsOff_shouldNotStoreEmptyColumns null ), new ParallelIndexIOConfig( - null, getInputSource(), JSON_FORMAT, false, diff --git a/indexing-service/src/test/java/org/apache/druid/indexing/common/task/batch/parallel/ParallelIndexSupervisorTaskKillTest.java b/indexing-service/src/test/java/org/apache/druid/indexing/common/task/batch/parallel/ParallelIndexSupervisorTaskKillTest.java index e85df4fe5dd4..cbf711469734 100644 --- a/indexing-service/src/test/java/org/apache/druid/indexing/common/task/batch/parallel/ParallelIndexSupervisorTaskKillTest.java +++ b/indexing-service/src/test/java/org/apache/druid/indexing/common/task/batch/parallel/ParallelIndexSupervisorTaskKillTest.java @@ -76,7 +76,6 @@ public void testStopGracefully() throws Exception final ParallelIndexSupervisorTask task = newTask( Intervals.of("2017/2018"), new ParallelIndexIOConfig( - null, // Sub-tasks would run forever new TestInputSource(Pair.of(new TestInput(Integer.MAX_VALUE, TaskState.SUCCESS), 4)), new NoopInputFormat(), @@ -103,7 +102,6 @@ public void testSubTaskFail() throws Exception final ParallelIndexSupervisorTask task = newTask( Intervals.of("2017/2018"), new ParallelIndexIOConfig( - null, new TestInputSource( Pair.of(new TestInput(10L, TaskState.FAILED), 1), Pair.of(new TestInput(Integer.MAX_VALUE, TaskState.FAILED), 3) @@ -282,7 +280,6 @@ SinglePhaseSubTaskSpec newTaskSpec(InputSplit split) new ParallelIndexIngestionSpec( getIngestionSchema().getDataSchema(), new ParallelIndexIOConfig( - null, baseInputSource.withSplit(split), getIngestionSchema().getIOConfig().getInputFormat(), getIngestionSchema().getIOConfig().isAppendToExisting(), diff --git a/indexing-service/src/test/java/org/apache/druid/indexing/common/task/batch/parallel/ParallelIndexSupervisorTaskResourceTest.java b/indexing-service/src/test/java/org/apache/druid/indexing/common/task/batch/parallel/ParallelIndexSupervisorTaskResourceTest.java index 50939a99d8a1..c9858b80847b 100644 --- a/indexing-service/src/test/java/org/apache/druid/indexing/common/task/batch/parallel/ParallelIndexSupervisorTaskResourceTest.java +++ b/indexing-service/src/test/java/org/apache/druid/indexing/common/task/batch/parallel/ParallelIndexSupervisorTaskResourceTest.java @@ -131,7 +131,6 @@ public void testAPIs() throws Exception task = newTask( Intervals.of("2017/2018"), new ParallelIndexIOConfig( - null, new TestInputSource(IntStream.range(0, NUM_SUB_TASKS).boxed().collect(Collectors.toList())), new NoopInputFormat(), false, @@ -519,7 +518,6 @@ SinglePhaseSubTaskSpec newTaskSpec(InputSplit split) new ParallelIndexIngestionSpec( getIngestionSchema().getDataSchema(), new ParallelIndexIOConfig( - null, baseInputSource.withSplit(split), getIngestionSchema().getIOConfig().getInputFormat(), getIngestionSchema().getIOConfig().isAppendToExisting(), diff --git a/indexing-service/src/test/java/org/apache/druid/indexing/common/task/batch/parallel/ParallelIndexSupervisorTaskSerdeTest.java b/indexing-service/src/test/java/org/apache/druid/indexing/common/task/batch/parallel/ParallelIndexSupervisorTaskSerdeTest.java index cd588d0b6eea..e11fb2ef001c 100644 --- a/indexing-service/src/test/java/org/apache/druid/indexing/common/task/batch/parallel/ParallelIndexSupervisorTaskSerdeTest.java +++ b/indexing-service/src/test/java/org/apache/druid/indexing/common/task/batch/parallel/ParallelIndexSupervisorTaskSerdeTest.java @@ -182,7 +182,6 @@ private static class ParallelIndexIngestionSpecBuilder ); private final ParallelIndexIOConfig ioConfig = new ParallelIndexIOConfig( - null, new LocalInputSource(new File("tmp"), "test_*"), new CsvInputFormat(Arrays.asList("ts", "dim", "val"), null, null, false, 0), false, diff --git a/indexing-service/src/test/java/org/apache/druid/indexing/common/task/batch/parallel/ParallelIndexSupervisorTaskTest.java b/indexing-service/src/test/java/org/apache/druid/indexing/common/task/batch/parallel/ParallelIndexSupervisorTaskTest.java index 699476dbc16d..57dbafa173f9 100644 --- a/indexing-service/src/test/java/org/apache/druid/indexing/common/task/batch/parallel/ParallelIndexSupervisorTaskTest.java +++ b/indexing-service/src/test/java/org/apache/druid/indexing/common/task/batch/parallel/ParallelIndexSupervisorTaskTest.java @@ -231,7 +231,6 @@ public void testFailToConstructWhenBothAppendToExistingAndForceGuaranteedRollupA final boolean appendToExisting = true; final boolean forceGuaranteedRollup = true; final ParallelIndexIOConfig ioConfig = new ParallelIndexIOConfig( - null, new InlineInputSource("test"), new JsonInputFormat(null, null, null, null, null), appendToExisting, @@ -291,7 +290,6 @@ public void testFailToConstructWhenBothInputSourceAndParserAreSet() { final ObjectMapper mapper = new DefaultObjectMapper(); final ParallelIndexIOConfig ioConfig = new ParallelIndexIOConfig( - null, new InlineInputSource("test"), null, false, @@ -528,7 +526,6 @@ public void testCompactionTaskDoesntCleanup() throws Exception final boolean appendToExisting = false; final boolean forceGuaranteedRollup = true; final ParallelIndexIOConfig ioConfig = new ParallelIndexIOConfig( - null, new InlineInputSource("test"), new JsonInputFormat(null, null, null, null, null), appendToExisting, diff --git a/indexing-service/src/test/java/org/apache/druid/indexing/common/task/batch/parallel/ParallelIndexTestingFactory.java b/indexing-service/src/test/java/org/apache/druid/indexing/common/task/batch/parallel/ParallelIndexTestingFactory.java index 16976e827d23..2455ce692b95 100644 --- a/indexing-service/src/test/java/org/apache/druid/indexing/common/task/batch/parallel/ParallelIndexTestingFactory.java +++ b/indexing-service/src/test/java/org/apache/druid/indexing/common/task/batch/parallel/ParallelIndexTestingFactory.java @@ -116,7 +116,7 @@ static ParallelIndexIngestionSpec createIngestionSpec( DataSchema dataSchema ) { - ParallelIndexIOConfig ioConfig = new ParallelIndexIOConfig(null, inputSource, inputFormat, false, false); + ParallelIndexIOConfig ioConfig = new ParallelIndexIOConfig(inputSource, inputFormat, false, false); return new ParallelIndexIngestionSpec(dataSchema, ioConfig, tuningConfig); } diff --git a/indexing-service/src/test/java/org/apache/druid/indexing/common/task/batch/parallel/RangePartitionTaskKillTest.java b/indexing-service/src/test/java/org/apache/druid/indexing/common/task/batch/parallel/RangePartitionTaskKillTest.java index 97e4f54c06ef..96494b8ac794 100644 --- a/indexing-service/src/test/java/org/apache/druid/indexing/common/task/batch/parallel/RangePartitionTaskKillTest.java +++ b/indexing-service/src/test/java/org/apache/druid/indexing/common/task/batch/parallel/RangePartitionTaskKillTest.java @@ -323,7 +323,6 @@ protected ParallelIndexSupervisorTask newTask( Preconditions.checkArgument(parseSpec == null); ParallelIndexIOConfig ioConfig = new ParallelIndexIOConfig( - null, new LocalInputSource(inputDir, filter), inputFormat, appendToExisting, diff --git a/indexing-service/src/test/java/org/apache/druid/indexing/common/task/batch/parallel/SinglePhaseParallelIndexingTest.java b/indexing-service/src/test/java/org/apache/druid/indexing/common/task/batch/parallel/SinglePhaseParallelIndexingTest.java index de060e521a9f..ac8371acaa0d 100644 --- a/indexing-service/src/test/java/org/apache/druid/indexing/common/task/batch/parallel/SinglePhaseParallelIndexingTest.java +++ b/indexing-service/src/test/java/org/apache/druid/indexing/common/task/batch/parallel/SinglePhaseParallelIndexingTest.java @@ -406,7 +406,6 @@ public void testRunInParallelIngestNullColumn() null ), new ParallelIndexIOConfig( - null, new SettableSplittableLocalInputSource(inputDir, VALID_INPUT_SOURCE_FILTER, true), DEFAULT_INPUT_FORMAT, false, @@ -460,7 +459,6 @@ public void testRunInParallelIngestNullColumn_storeEmptyColumnsOff_shouldNotStor null ), new ParallelIndexIOConfig( - null, new SettableSplittableLocalInputSource(inputDir, VALID_INPUT_SOURCE_FILTER, true), DEFAULT_INPUT_FORMAT, false, @@ -803,7 +801,6 @@ public void testIngestBothExplicitAndImplicitDims() throws IOException null ), new ParallelIndexIOConfig( - null, new SettableSplittableLocalInputSource(inputDir, "*.json", true), new JsonInputFormat( new JSONPathSpec(true, null), @@ -887,7 +884,6 @@ public void testIngestBothExplicitAndImplicitDimsSchemaDiscovery() throws IOExce null ), new ParallelIndexIOConfig( - null, new SettableSplittableLocalInputSource(inputDir, "*.json", true), new JsonInputFormat( new JSONPathSpec(true, null), @@ -967,7 +963,6 @@ private ParallelIndexSupervisorTask newTask( null ), new ParallelIndexIOConfig( - null, new SettableSplittableLocalInputSource(inputDir, inputSourceFilter, splittableInputSource), DEFAULT_INPUT_FORMAT, appendToExisting, @@ -990,7 +985,6 @@ private ParallelIndexSupervisorTask newTask( null ), new ParallelIndexIOConfig( - null, new LocalInputSource(inputDir, inputSourceFilter), createInputFormatFromParseSpec(DEFAULT_PARSE_SPEC), appendToExisting, diff --git a/indexing-service/src/test/java/org/apache/druid/indexing/common/task/batch/parallel/SinglePhaseSubTaskSpecTest.java b/indexing-service/src/test/java/org/apache/druid/indexing/common/task/batch/parallel/SinglePhaseSubTaskSpecTest.java index b70cc7a99e69..4acc3d3f5912 100644 --- a/indexing-service/src/test/java/org/apache/druid/indexing/common/task/batch/parallel/SinglePhaseSubTaskSpecTest.java +++ b/indexing-service/src/test/java/org/apache/druid/indexing/common/task/batch/parallel/SinglePhaseSubTaskSpecTest.java @@ -57,7 +57,6 @@ public class SinglePhaseSubTaskSpecTest null ), new ParallelIndexIOConfig( - null, new LocalInputSource(new File("baseDir"), "filter"), new JsonInputFormat(null, null, null, null, null), null, diff --git a/indexing-service/src/test/java/org/apache/druid/indexing/input/DruidInputSourceTest.java b/indexing-service/src/test/java/org/apache/druid/indexing/input/DruidInputSourceTest.java index 6c6db80c1c72..4fc77934f60e 100644 --- a/indexing-service/src/test/java/org/apache/druid/indexing/input/DruidInputSourceTest.java +++ b/indexing-service/src/test/java/org/apache/druid/indexing/input/DruidInputSourceTest.java @@ -34,7 +34,6 @@ import org.apache.druid.indexing.common.RetryPolicyFactory; import org.apache.druid.indexing.common.SegmentCacheManagerFactory; import org.apache.druid.indexing.common.config.TaskConfig; -import org.apache.druid.indexing.firehose.WindowedSegmentId; import org.apache.druid.java.util.common.Intervals; import org.apache.druid.segment.IndexIO; import org.apache.druid.segment.TestHelper; diff --git a/indexing-service/src/test/java/org/apache/druid/indexing/firehose/WindowedSegmentIdTest.java b/indexing-service/src/test/java/org/apache/druid/indexing/input/WindowedSegmentIdTest.java similarity index 96% rename from indexing-service/src/test/java/org/apache/druid/indexing/firehose/WindowedSegmentIdTest.java rename to indexing-service/src/test/java/org/apache/druid/indexing/input/WindowedSegmentIdTest.java index 0586e98c11ba..add3ccc0a3cb 100644 --- a/indexing-service/src/test/java/org/apache/druid/indexing/firehose/WindowedSegmentIdTest.java +++ b/indexing-service/src/test/java/org/apache/druid/indexing/input/WindowedSegmentIdTest.java @@ -17,7 +17,7 @@ * under the License. */ -package org.apache.druid.indexing.firehose; +package org.apache.druid.indexing.input; import nl.jqno.equalsverifier.EqualsVerifier; import org.junit.Test; diff --git a/indexing-service/src/test/java/org/apache/druid/indexing/overlord/ForkingTaskRunnerTest.java b/indexing-service/src/test/java/org/apache/druid/indexing/overlord/ForkingTaskRunnerTest.java index 56695e5cf27c..708e690db6c5 100644 --- a/indexing-service/src/test/java/org/apache/druid/indexing/overlord/ForkingTaskRunnerTest.java +++ b/indexing-service/src/test/java/org/apache/druid/indexing/overlord/ForkingTaskRunnerTest.java @@ -389,7 +389,6 @@ public void testJavaOptsAndJavaOptsArrayOverride() throws ExecutionException, In + " \"runTime\" : 2500,\n" + " \"isReadyTime\" : 0,\n" + " \"isReadyResult\" : \"YES\",\n" - + " \"firehose\" : null,\n" + " \"context\" : {\n" + " \"druid.indexer.runner.javaOptsArray\" : [ \"-Xmx10g\", \"-Xms10g\" ],\n" + " \"druid.indexer.fork.property.druid.processing.numThreads\" : 4,\n" @@ -447,7 +446,6 @@ public void testInvalidTaskContextJavaOptsArray() throws JsonProcessingException + " \"runTime\" : 2500,\n" + " \"isReadyTime\" : 0,\n" + " \"isReadyResult\" : \"YES\",\n" - + " \"firehose\" : null,\n" + " \"context\" : {\n" + " \"druid.indexer.runner.javaOptsArray\" : \"not a string array\",\n" + " \"druid.indexer.runner.javaOpts\" : \"-Xmx1g -Xms1g\"\n" diff --git a/indexing-service/src/test/java/org/apache/druid/indexing/overlord/SingleTaskBackgroundRunnerTest.java b/indexing-service/src/test/java/org/apache/druid/indexing/overlord/SingleTaskBackgroundRunnerTest.java index ad34aa10ebe9..5ee6ac6f4c17 100644 --- a/indexing-service/src/test/java/org/apache/druid/indexing/overlord/SingleTaskBackgroundRunnerTest.java +++ b/indexing-service/src/test/java/org/apache/druid/indexing/overlord/SingleTaskBackgroundRunnerTest.java @@ -52,7 +52,7 @@ import org.apache.druid.segment.loading.NoopDataSegmentMover; import org.apache.druid.segment.loading.NoopDataSegmentPusher; import org.apache.druid.segment.metadata.CentralizedDatasourceSchemaConfig; -import org.apache.druid.segment.realtime.firehose.NoopChatHandlerProvider; +import org.apache.druid.segment.realtime.NoopChatHandlerProvider; import org.apache.druid.server.DruidNode; import org.apache.druid.server.SetAndVerifyContextQueryRunner; import org.apache.druid.server.coordination.NoopDataSegmentAnnouncer; diff --git a/indexing-service/src/test/java/org/apache/druid/indexing/overlord/TaskLifecycleTest.java b/indexing-service/src/test/java/org/apache/druid/indexing/overlord/TaskLifecycleTest.java index 15eb216d0bbe..2df65120d350 100644 --- a/indexing-service/src/test/java/org/apache/druid/indexing/overlord/TaskLifecycleTest.java +++ b/indexing-service/src/test/java/org/apache/druid/indexing/overlord/TaskLifecycleTest.java @@ -37,8 +37,6 @@ import org.apache.druid.client.coordinator.NoopCoordinatorClient; import org.apache.druid.client.indexing.NoopOverlordClient; import org.apache.druid.data.input.AbstractInputSource; -import org.apache.druid.data.input.Firehose; -import org.apache.druid.data.input.FirehoseFactory; import org.apache.druid.data.input.InputRow; import org.apache.druid.data.input.InputRowListPlusRawValues; import org.apache.druid.data.input.InputRowSchema; @@ -46,7 +44,6 @@ import org.apache.druid.data.input.InputStats; import org.apache.druid.data.input.MapBasedInputRow; import org.apache.druid.data.input.impl.DimensionsSpec; -import org.apache.druid.data.input.impl.InputRowParser; import org.apache.druid.data.input.impl.NoopInputFormat; import org.apache.druid.data.input.impl.TimestampSpec; import org.apache.druid.discovery.DataNodeService; @@ -130,9 +127,9 @@ import org.apache.druid.segment.loading.LocalDataSegmentPusherConfig; import org.apache.druid.segment.loading.NoopDataSegmentArchiver; import org.apache.druid.segment.metadata.CentralizedDatasourceSchemaConfig; +import org.apache.druid.segment.realtime.NoopChatHandlerProvider; import org.apache.druid.segment.realtime.appenderator.AppenderatorsManager; import org.apache.druid.segment.realtime.appenderator.UnifiedIndexerAppenderatorsManager; -import org.apache.druid.segment.realtime.firehose.NoopChatHandlerProvider; import org.apache.druid.server.DruidNode; import org.apache.druid.server.coordination.DataSegmentServerAnnouncer; import org.apache.druid.server.coordination.ServerType; @@ -368,37 +365,6 @@ public boolean needsFormat() } } - private static class MockFirehoseFactory implements FirehoseFactory - { - @Override - public Firehose connect(InputRowParser parser, File temporaryDirectory) - { - final Iterator inputRowIterator = REALTIME_IDX_TASK_INPUT_ROWS.iterator(); - - return new Firehose() - { - @Override - public boolean hasMore() - { - return inputRowIterator.hasNext(); - } - - @Nullable - @Override - public InputRow nextRow() - { - return inputRowIterator.next(); - } - - @Override - public void close() - { - - } - }; - } - } - @Before public void setUp() throws Exception { @@ -446,7 +412,6 @@ private TaskStorage setUpTaskStorage() case METADATA_TASK_STORAGE: { TestDerbyConnector testDerbyConnector = derbyConnectorRule.getConnector(); mapper.registerSubtypes( - new NamedType(MockFirehoseFactory.class, "mockFirehoseFactory"), new NamedType(MockInputSource.class, "mockInputSource"), new NamedType(NoopInputFormat.class, "noopInputFormat") ); @@ -720,7 +685,7 @@ public void testIndexTask() ), null ), - new IndexIOConfig(null, new MockInputSource(), new NoopInputFormat(), false, false), + new IndexIOConfig(new MockInputSource(), new NoopInputFormat(), false, false), TuningConfigBuilder.forIndexTask() .withMaxRowsPerSegment(10000) .withMaxRowsInMemory(100) @@ -783,7 +748,7 @@ public void testIndexTaskFailure() null, mapper ), - new IndexIOConfig(null, new MockExceptionInputSource(), new NoopInputFormat(), false, false), + new IndexIOConfig(new MockExceptionInputSource(), new NoopInputFormat(), false, false), TuningConfigBuilder.forIndexTask() .withMaxRowsPerSegment(10000) .withMaxRowsInMemory(10) @@ -1213,7 +1178,7 @@ public void testResumeTasks() throws Exception ), null ), - new IndexIOConfig(null, new MockInputSource(), new NoopInputFormat(), false, false), + new IndexIOConfig(new MockInputSource(), new NoopInputFormat(), false, false), TuningConfigBuilder.forIndexTask() .withMaxRowsPerSegment(10000) .withMaxRowsInMemory(10) @@ -1301,7 +1266,7 @@ public void testUnifiedAppenderatorsManagerCleanup() throws Exception ), null ), - new IndexIOConfig(null, new MockInputSource(), new NoopInputFormat(), false, false), + new IndexIOConfig(new MockInputSource(), new NoopInputFormat(), false, false), TuningConfigBuilder.forIndexTask() .withMaxRowsPerSegment(10000) .withMaxRowsInMemory(10) diff --git a/indexing-service/src/test/java/org/apache/druid/indexing/overlord/TaskQueueTest.java b/indexing-service/src/test/java/org/apache/druid/indexing/overlord/TaskQueueTest.java index 8984e73b6b9c..ee90a3335a17 100644 --- a/indexing-service/src/test/java/org/apache/druid/indexing/overlord/TaskQueueTest.java +++ b/indexing-service/src/test/java/org/apache/druid/indexing/overlord/TaskQueueTest.java @@ -558,7 +558,6 @@ public void testGetActiveTaskRedactsPassword() throws JsonProcessingException null ); final ParallelIndexIOConfig ioConfig = new ParallelIndexIOConfig( - null, new HttpInputSource(Collections.singletonList(URI.create("http://host.org")), "user", new DefaultPasswordProvider(password), diff --git a/indexing-service/src/test/java/org/apache/druid/indexing/overlord/TestTaskToolboxFactory.java b/indexing-service/src/test/java/org/apache/druid/indexing/overlord/TestTaskToolboxFactory.java index edebc0fe8dc8..19e4bd33fd8e 100644 --- a/indexing-service/src/test/java/org/apache/druid/indexing/overlord/TestTaskToolboxFactory.java +++ b/indexing-service/src/test/java/org/apache/druid/indexing/overlord/TestTaskToolboxFactory.java @@ -57,8 +57,8 @@ import org.apache.druid.segment.loading.DataSegmentMover; import org.apache.druid.segment.loading.DataSegmentPusher; import org.apache.druid.segment.metadata.CentralizedDatasourceSchemaConfig; +import org.apache.druid.segment.realtime.ChatHandlerProvider; import org.apache.druid.segment.realtime.appenderator.AppenderatorsManager; -import org.apache.druid.segment.realtime.firehose.ChatHandlerProvider; import org.apache.druid.segment.writeout.OnHeapMemorySegmentWriteOutMediumFactory; import org.apache.druid.server.DruidNode; import org.apache.druid.server.coordination.DataSegmentAnnouncer; diff --git a/indexing-service/src/test/java/org/apache/druid/indexing/overlord/http/OverlordResourceTest.java b/indexing-service/src/test/java/org/apache/druid/indexing/overlord/http/OverlordResourceTest.java index 687d1deb7b15..e6dee0c7e403 100644 --- a/indexing-service/src/test/java/org/apache/druid/indexing/overlord/http/OverlordResourceTest.java +++ b/indexing-service/src/test/java/org/apache/druid/indexing/overlord/http/OverlordResourceTest.java @@ -1423,7 +1423,7 @@ public void testResourceActionsForTaskWithInputTypeAndInputSecurityEnabled() } @Test - public void testResourceActionsForTaskWithFirehoseAndInputSecurityEnabled() + public void testResourceActionsForTaskWithInvalidSecurityAndInputSecurityEnabled() { final String dataSource = "dataSourceTest"; final UOE expectedException = new UOE("unsupported"); diff --git a/indexing-service/src/test/java/org/apache/druid/indexing/overlord/sampler/IndexTaskSamplerSpecTest.java b/indexing-service/src/test/java/org/apache/druid/indexing/overlord/sampler/IndexTaskSamplerSpecTest.java index 6f92aeef646b..36dfd966e8d2 100644 --- a/indexing-service/src/test/java/org/apache/druid/indexing/overlord/sampler/IndexTaskSamplerSpecTest.java +++ b/indexing-service/src/test/java/org/apache/druid/indexing/overlord/sampler/IndexTaskSamplerSpecTest.java @@ -20,7 +20,6 @@ package org.apache.druid.indexing.overlord.sampler; import com.fasterxml.jackson.databind.InjectableValues; -import com.fasterxml.jackson.databind.Module; import com.fasterxml.jackson.databind.ObjectMapper; import com.google.common.collect.ImmutableList; import org.apache.druid.client.indexing.SamplerResponse; @@ -28,7 +27,6 @@ import org.apache.druid.data.input.InputSource; import org.apache.druid.data.input.impl.JsonInputFormat; import org.apache.druid.data.input.impl.LocalInputSource; -import org.apache.druid.guice.FirehoseModule; import org.apache.druid.segment.TestHelper; import org.apache.druid.segment.column.RowSignature; import org.apache.druid.segment.indexing.DataSchema; @@ -64,8 +62,7 @@ public IndexTaskSamplerSpecTest() .addValue(InputSourceSampler.class, inputSourceSampler) .addValue(ObjectMapper.class, MAPPER) ); - MAPPER.registerModules((Iterable) new SamplerModule().getJacksonModules()); - MAPPER.registerModules((Iterable) new FirehoseModule().getJacksonModules()); + MAPPER.registerModules(new SamplerModule().getJacksonModules()); } @Test diff --git a/indexing-service/src/test/java/org/apache/druid/indexing/seekablestream/SeekableStreamIndexTaskTestBase.java b/indexing-service/src/test/java/org/apache/druid/indexing/seekablestream/SeekableStreamIndexTaskTestBase.java index 3b5775c22b83..ea6d97448f8a 100644 --- a/indexing-service/src/test/java/org/apache/druid/indexing/seekablestream/SeekableStreamIndexTaskTestBase.java +++ b/indexing-service/src/test/java/org/apache/druid/indexing/seekablestream/SeekableStreamIndexTaskTestBase.java @@ -117,8 +117,8 @@ import org.apache.druid.segment.loading.LocalDataSegmentPusherConfig; import org.apache.druid.segment.metadata.CentralizedDatasourceSchemaConfig; import org.apache.druid.segment.metadata.SegmentSchemaManager; +import org.apache.druid.segment.realtime.NoopChatHandlerProvider; import org.apache.druid.segment.realtime.appenderator.StreamAppenderator; -import org.apache.druid.segment.realtime.firehose.NoopChatHandlerProvider; import org.apache.druid.server.DruidNode; import org.apache.druid.server.coordination.DataSegmentServerAnnouncer; import org.apache.druid.server.coordination.ServerType; diff --git a/indexing-service/src/test/java/org/apache/druid/indexing/worker/TaskAnnouncementTest.java b/indexing-service/src/test/java/org/apache/druid/indexing/worker/TaskAnnouncementTest.java index ca3cd6edf8b3..88249509ef97 100644 --- a/indexing-service/src/test/java/org/apache/druid/indexing/worker/TaskAnnouncementTest.java +++ b/indexing-service/src/test/java/org/apache/druid/indexing/worker/TaskAnnouncementTest.java @@ -46,7 +46,7 @@ public TaskAnnouncementTest() @Test public void testBackwardsCompatibleSerde() throws Exception { - final IndexTask.IndexIOConfig ioConfig = new IndexTask.IndexIOConfig(null, new NoopInputSource(), null, null, null); + final IndexTask.IndexIOConfig ioConfig = new IndexTask.IndexIOConfig(new NoopInputSource(), null, null, null); final Task task = new IndexTask( "theid", new TaskResource("rofl", 2), diff --git a/indexing-service/src/test/java/org/apache/druid/indexing/worker/WorkerTaskManagerTest.java b/indexing-service/src/test/java/org/apache/druid/indexing/worker/WorkerTaskManagerTest.java index 8217a12eb837..3eea9d33fb3e 100644 --- a/indexing-service/src/test/java/org/apache/druid/indexing/worker/WorkerTaskManagerTest.java +++ b/indexing-service/src/test/java/org/apache/druid/indexing/worker/WorkerTaskManagerTest.java @@ -52,7 +52,7 @@ import org.apache.druid.segment.handoff.SegmentHandoffNotifierFactory; import org.apache.druid.segment.join.NoopJoinableFactory; import org.apache.druid.segment.metadata.CentralizedDatasourceSchemaConfig; -import org.apache.druid.segment.realtime.firehose.NoopChatHandlerProvider; +import org.apache.druid.segment.realtime.NoopChatHandlerProvider; import org.apache.druid.server.coordination.ChangeRequestHistory; import org.apache.druid.server.coordination.ChangeRequestsSnapshot; import org.apache.druid.server.security.AuthTestUtils; diff --git a/indexing-service/src/test/java/org/apache/druid/indexing/worker/WorkerTaskMonitorTest.java b/indexing-service/src/test/java/org/apache/druid/indexing/worker/WorkerTaskMonitorTest.java index c3191cc068d5..fcdada15f5da 100644 --- a/indexing-service/src/test/java/org/apache/druid/indexing/worker/WorkerTaskMonitorTest.java +++ b/indexing-service/src/test/java/org/apache/druid/indexing/worker/WorkerTaskMonitorTest.java @@ -55,7 +55,7 @@ import org.apache.druid.segment.handoff.SegmentHandoffNotifierFactory; import org.apache.druid.segment.join.NoopJoinableFactory; import org.apache.druid.segment.metadata.CentralizedDatasourceSchemaConfig; -import org.apache.druid.segment.realtime.firehose.NoopChatHandlerProvider; +import org.apache.druid.segment.realtime.NoopChatHandlerProvider; import org.apache.druid.server.DruidNode; import org.apache.druid.server.initialization.IndexerZkConfig; import org.apache.druid.server.initialization.ServerConfig; diff --git a/integration-tests-ex/cases/src/test/java/org/apache/druid/testsEx/indexer/ITBestEffortRollupParallelIndexTest.java b/integration-tests-ex/cases/src/test/java/org/apache/druid/testsEx/indexer/ITBestEffortRollupParallelIndexTest.java index 96e8d500fa6e..20bdd9767a77 100644 --- a/integration-tests-ex/cases/src/test/java/org/apache/druid/testsEx/indexer/ITBestEffortRollupParallelIndexTest.java +++ b/integration-tests-ex/cases/src/test/java/org/apache/druid/testsEx/indexer/ITBestEffortRollupParallelIndexTest.java @@ -130,7 +130,7 @@ public void testIndexData() throws Exception new Pair<>(false, false) ); - // with DruidInputSource instead of IngestSegmentFirehose + // with DruidInputSource doReindexTest( INDEX_DATASOURCE, INDEX_DRUID_INPUT_SOURCE_DATASOURCE, diff --git a/integration-tests/src/main/java/org/apache/druid/testing/clients/EventReceiverFirehoseTestClient.java b/integration-tests/src/main/java/org/apache/druid/testing/clients/EventReceiverFirehoseTestClient.java deleted file mode 100644 index 761d9f55dbaf..000000000000 --- a/integration-tests/src/main/java/org/apache/druid/testing/clients/EventReceiverFirehoseTestClient.java +++ /dev/null @@ -1,181 +0,0 @@ -/* - * Licensed to the Apache Software Foundation (ASF) under one - * or more contributor license agreements. See the NOTICE file - * distributed with this work for additional information - * regarding copyright ownership. The ASF licenses this file - * to you under the Apache License, Version 2.0 (the - * "License"); you may not use this file except in compliance - * with the License. You may obtain a copy of the License at - * - * http://www.apache.org/licenses/LICENSE-2.0 - * - * Unless required by applicable law or agreed to in writing, - * software distributed under the License is distributed on an - * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY - * KIND, either express or implied. See the License for the - * specific language governing permissions and limitations - * under the License. - */ - -package org.apache.druid.testing.clients; - -import com.fasterxml.jackson.core.type.TypeReference; -import com.fasterxml.jackson.databind.ObjectMapper; -import com.fasterxml.jackson.jaxrs.smile.SmileMediaTypes; -import org.apache.druid.java.util.common.ISE; -import org.apache.druid.java.util.common.StringUtils; -import org.apache.druid.java.util.common.jackson.JacksonUtils; -import org.apache.druid.java.util.common.logger.Logger; -import org.apache.druid.java.util.http.client.HttpClient; -import org.apache.druid.java.util.http.client.Request; -import org.apache.druid.java.util.http.client.response.StatusResponseHandler; -import org.apache.druid.java.util.http.client.response.StatusResponseHolder; -import org.apache.druid.testing.guice.TestClient; -import org.jboss.netty.handler.codec.http.HttpMethod; -import org.jboss.netty.handler.codec.http.HttpResponseStatus; - -import javax.ws.rs.core.MediaType; -import java.io.BufferedReader; -import java.io.InputStreamReader; -import java.net.URL; -import java.nio.charset.StandardCharsets; -import java.util.ArrayList; -import java.util.Collection; -import java.util.Map; -import java.util.concurrent.ExecutionException; - -public class EventReceiverFirehoseTestClient -{ - private static final Logger LOG = new Logger(EventReceiverFirehoseTestClient.class); - - static final int NUM_RETRIES = 30; - static final long DELAY_FOR_RETRIES_MS = 10000; - - private final String host; - private final ObjectMapper jsonMapper; - private final HttpClient httpClient; - private final String chatID; - private final ObjectMapper smileMapper; - - public EventReceiverFirehoseTestClient( - String host, - String chatID, - ObjectMapper jsonMapper, - @TestClient HttpClient httpClient, - ObjectMapper smileMapper - ) - { - this.host = host; - this.jsonMapper = jsonMapper; - this.httpClient = httpClient; - this.chatID = chatID; - this.smileMapper = smileMapper; - } - - private String getURL() - { - return StringUtils.format( - "https://%s/druid/worker/v1/chat/%s/push-events/", - host, - chatID - ); - } - - /** - * post events from the collection and return the count of events accepted - * - * @param events Collection of events to be posted - * - * @return - */ - public int postEvents(Collection> events, ObjectMapper objectMapper, String mediaType) - throws InterruptedException - { - int retryCount = 0; - while (true) { - try { - StatusResponseHolder response = httpClient.go( - new Request(HttpMethod.POST, new URL(getURL())) - .setContent(mediaType, objectMapper.writeValueAsBytes(events)), - StatusResponseHandler.getInstance() - ).get(); - - if (!response.getStatus().equals(HttpResponseStatus.OK)) { - throw new ISE( - "Error while posting events to url[%s] status[%s] content[%s]", - getURL(), - response.getStatus(), - response.getContent() - ); - } - Map responseData = objectMapper.readValue( - response.getContent(), new TypeReference>() - { - } - ); - return responseData.get("eventCount"); - } - // adding retries to flaky tests using channels - catch (ExecutionException e) { - if (retryCount > NUM_RETRIES) { - throw new RuntimeException(e); //giving up now - } else { - LOG.info(e, "received exception, sleeping and retrying"); - retryCount++; - Thread.sleep(DELAY_FOR_RETRIES_MS); - } - } - catch (Exception e) { - throw new RuntimeException(e); - } - } - } - - /** - * Reads each events from file and post them to the indexing service. - * Uses both smileMapper and jsonMapper to send events alternately. - * - * @param file location of file to post events from - * - * @return number of events sent to the indexing service - */ - public int postEventsFromFile(String file) - { - try ( - BufferedReader reader = new BufferedReader( - new InputStreamReader( - EventReceiverFirehoseTestClient.class.getResourceAsStream(file), - StandardCharsets.UTF_8 - ) - ) - ) { - - String s; - Collection> events = new ArrayList>(); - // Test sending events using both jsonMapper and smileMapper. - // sends events one by one using both jsonMapper and smileMapper. - int totalEventsPosted = 0; - int expectedEventsPosted = 0; - while ((s = reader.readLine()) != null) { - events.add(this.jsonMapper.readValue(s, JacksonUtils.TYPE_REFERENCE_MAP_STRING_OBJECT)); - ObjectMapper mapper = (totalEventsPosted % 2 == 0) ? jsonMapper : smileMapper; - String mediaType = (totalEventsPosted % 2 == 0) - ? MediaType.APPLICATION_JSON - : SmileMediaTypes.APPLICATION_JACKSON_SMILE; - totalEventsPosted += postEvents(events, mapper, mediaType); - - expectedEventsPosted += events.size(); - events = new ArrayList<>(); - } - - if (totalEventsPosted != expectedEventsPosted) { - throw new ISE("All events not posted, expected : %d actual : %d", events.size(), totalEventsPosted); - } - return totalEventsPosted; - } - catch (Exception e) { - throw new RuntimeException(e); - } - - } -} diff --git a/integration-tests/src/main/java/org/apache/druid/testing/guice/DruidTestModuleFactory.java b/integration-tests/src/main/java/org/apache/druid/testing/guice/DruidTestModuleFactory.java index d08712dea35e..1195795cd970 100644 --- a/integration-tests/src/main/java/org/apache/druid/testing/guice/DruidTestModuleFactory.java +++ b/integration-tests/src/main/java/org/apache/druid/testing/guice/DruidTestModuleFactory.java @@ -23,7 +23,6 @@ import com.google.inject.Injector; import com.google.inject.Module; import org.apache.druid.guice.GuiceInjectors; -import org.apache.druid.guice.IndexingServiceFirehoseModule; import org.apache.druid.guice.IndexingServiceInputSourceModule; import org.apache.druid.guice.IndexingServiceTuningConfigModule; import org.apache.druid.initialization.Initialization; @@ -50,7 +49,6 @@ private static List getModules() { return ImmutableList.of( new DruidTestModule(), - new IndexingServiceFirehoseModule(), new IndexingServiceInputSourceModule(), new IndexingServiceTuningConfigModule() ); diff --git a/integration-tests/src/test/java/org/apache/druid/tests/indexer/ITBestEffortRollupParallelIndexTest.java b/integration-tests/src/test/java/org/apache/druid/tests/indexer/ITBestEffortRollupParallelIndexTest.java index be2ab61a5f59..1312685fe5a1 100644 --- a/integration-tests/src/test/java/org/apache/druid/tests/indexer/ITBestEffortRollupParallelIndexTest.java +++ b/integration-tests/src/test/java/org/apache/druid/tests/indexer/ITBestEffortRollupParallelIndexTest.java @@ -137,7 +137,7 @@ public void testIndexData(PartitionsSpec partitionsSpec) throws Exception new Pair<>(false, false) ); - // with DruidInputSource instead of IngestSegmentFirehose + // with DruidInputSource doReindexTest( INDEX_DATASOURCE, INDEX_DRUID_INPUT_SOURCE_DATASOURCE, diff --git a/integration-tests/src/test/java/org/apache/druid/tests/indexer/ITPerfectRollupParallelIndexTest.java b/integration-tests/src/test/java/org/apache/druid/tests/indexer/ITPerfectRollupParallelIndexTest.java index 0e8fc904949c..5695156fa9bf 100644 --- a/integration-tests/src/test/java/org/apache/druid/tests/indexer/ITPerfectRollupParallelIndexTest.java +++ b/integration-tests/src/test/java/org/apache/druid/tests/indexer/ITPerfectRollupParallelIndexTest.java @@ -115,7 +115,7 @@ public void testIndexData(PartitionsSpec partitionsSpec) throws Exception ); - // with DruidInputSource instead of IngestSegmentFirehose + // with DruidInputSource doReindexTest( INDEX_DATASOURCE, INDEX_DRUID_INPUT_SOURCE_DATASOURCE, diff --git a/processing/src/main/java/org/apache/druid/data/input/Firehose.java b/processing/src/main/java/org/apache/druid/data/input/Firehose.java deleted file mode 100644 index a9ff6f33159c..000000000000 --- a/processing/src/main/java/org/apache/druid/data/input/Firehose.java +++ /dev/null @@ -1,97 +0,0 @@ -/* - * Licensed to the Apache Software Foundation (ASF) under one - * or more contributor license agreements. See the NOTICE file - * distributed with this work for additional information - * regarding copyright ownership. The ASF licenses this file - * to you under the Apache License, Version 2.0 (the - * "License"); you may not use this file except in compliance - * with the License. You may obtain a copy of the License at - * - * http://www.apache.org/licenses/LICENSE-2.0 - * - * Unless required by applicable law or agreed to in writing, - * software distributed under the License is distributed on an - * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY - * KIND, either express or implied. See the License for the - * specific language governing permissions and limitations - * under the License. - */ - -package org.apache.druid.data.input; - -import org.apache.druid.guice.annotations.ExtensionPoint; -import org.apache.druid.java.util.common.parsers.ParseException; - -import javax.annotation.Nullable; -import java.io.Closeable; -import java.io.IOException; - -/** - * This is an interface that holds onto the stream of incoming data. Realtime data ingestion is built around this - * abstraction. - * - * This object acts a lot like an Iterator, but it doesn't extend the Iterator interface because it extends - * Closeable and it is very important that the {@link #close()} method doesn't get forgotten, which is easy to do if - * this gets passed around as an Iterator. Note that {@link #close()} doesn't cut the stream of rows for Firehose users - * immediately, but rather stops the supply of new rows into internal buffers. {@link #hasMore()} and {@link #nextRow()} - * are expected to operate for some time after (or concurrently with) {@link #close()} until the buffered events (if - * any) run out. - * - * Concurrency: - * The two methods {@link #hasMore()} and {@link #nextRow()} are all called from the same thread. - * {@link #close()} might be called concurrently from a thread different from the thread calling {@link #hasMore()} - * and {@link #nextRow()}. - *

- */ -@Deprecated -@ExtensionPoint -public interface Firehose extends Closeable -{ - /** - * Returns whether there are more rows to process. This is used to indicate that another item is immediately - * available via {@link #nextRow()}. Thus, if the stream is still available but there are no new messages on it, this - * call should block until a new message is available. - * - * If something happens such that the stream is no longer available, this should return false. - * - * @return true if and when there is another row available, false if the stream has dried up - */ - boolean hasMore() throws IOException; - - /** - * The next row available. Should only be called if hasMore returns true. - * The return value can be null which means the caller must skip this row. - * - * @return The next row - */ - @Nullable - InputRow nextRow() throws IOException; - - /** - * Returns an {@link InputRowListPlusRawValues} object containing the InputRow plus the raw, unparsed data corresponding to - * the next row available. Used in the sampler to provide the caller with information to assist in configuring a parse - * spec. If a ParseException is thrown by the parser, it should be caught and returned in the InputRowListPlusRawValues so - * we will be able to provide information on the raw row which failed to be parsed. Should only be called if hasMore - * returns true. - * - * @return an InputRowListPlusRawValues which may contain any of: an InputRow, map of the raw data, or a ParseException - */ - @Deprecated - default InputRowListPlusRawValues nextRowWithRaw() throws IOException - { - try { - return InputRowListPlusRawValues.of(nextRow(), null); - } - catch (ParseException e) { - return InputRowListPlusRawValues.of(null, e); - } - } - - /** - * Closes the "ingestion side" of the Firehose, potentially concurrently with calls to {@link #hasMore()} and {@link - * #nextRow()} being made from a different thread. {@link #hasMore()} and {@link #nextRow()} - * continue to work after close(), but since the ingestion side is closed rows will eventually run out. - */ - @Override - void close() throws IOException; -} diff --git a/processing/src/main/java/org/apache/druid/data/input/FirehoseFactory.java b/processing/src/main/java/org/apache/druid/data/input/FirehoseFactory.java deleted file mode 100644 index 6ec8657d435a..000000000000 --- a/processing/src/main/java/org/apache/druid/data/input/FirehoseFactory.java +++ /dev/null @@ -1,82 +0,0 @@ -/* - * Licensed to the Apache Software Foundation (ASF) under one - * or more contributor license agreements. See the NOTICE file - * distributed with this work for additional information - * regarding copyright ownership. The ASF licenses this file - * to you under the Apache License, Version 2.0 (the - * "License"); you may not use this file except in compliance - * with the License. You may obtain a copy of the License at - * - * http://www.apache.org/licenses/LICENSE-2.0 - * - * Unless required by applicable law or agreed to in writing, - * software distributed under the License is distributed on an - * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY - * KIND, either express or implied. See the License for the - * specific language governing permissions and limitations - * under the License. - */ - -package org.apache.druid.data.input; - -import com.fasterxml.jackson.annotation.JsonIgnore; -import com.fasterxml.jackson.annotation.JsonTypeInfo; -import org.apache.druid.data.input.impl.InputRowParser; -import org.apache.druid.guice.annotations.ExtensionPoint; -import org.apache.druid.java.util.common.parsers.ParseException; - -import javax.annotation.Nullable; -import java.io.File; -import java.io.IOException; - -/** - * FirehoseFactory creates a {@link Firehose} which is an interface holding onto the stream of incoming data. - * It currently provides two methods for creating a {@link Firehose} and their default implementations call each other - * for the backward compatibility. Implementations of this interface must implement one of these methods. - * - * This class is deprecated in favor of {@link InputSource} - */ -@Deprecated -@ExtensionPoint -@JsonTypeInfo(use = JsonTypeInfo.Id.NAME, property = "type") -public interface FirehoseFactory -{ - /** - * Initialization method that connects up the fire hose. If this method returns successfully it should be safe to - * call hasMore() on the returned Firehose (which might subsequently block). - *

- * If this method returns null, then any attempt to call hasMore(), nextRow() and close() on the return - * value will throw a surprising NPE. Throwing IOException on connection failure or runtime exception on - * invalid configuration is preferred over returning null. - * - * @param parser an input row parser - */ - @Deprecated - default Firehose connect(T parser) throws IOException, ParseException - { - return connect(parser, null); - } - - /** - * Initialization method that connects up the fire hose. If this method returns successfully it should be safe to - * call hasMore() on the returned Firehose (which might subsequently block). - *

- * If this method returns null, then any attempt to call hasMore(), nextRow() and close() on the return - * value will throw a surprising NPE. Throwing IOException on connection failure or runtime exception on - * invalid configuration is preferred over returning null. - * - * @param parser an input row parser - * @param temporaryDirectory a directory where temporary files are stored - */ - default Firehose connect(T parser, @Nullable File temporaryDirectory) throws IOException, ParseException - { - return connect(parser); - } - - @SuppressWarnings("unused") - @JsonIgnore - default boolean isSplittable() - { - return false; - } -} diff --git a/processing/src/main/java/org/apache/druid/data/input/SegmentsSplitHintSpec.java b/processing/src/main/java/org/apache/druid/data/input/SegmentsSplitHintSpec.java index ed59d67285a6..433a6a2f9958 100644 --- a/processing/src/main/java/org/apache/druid/data/input/SegmentsSplitHintSpec.java +++ b/processing/src/main/java/org/apache/druid/data/input/SegmentsSplitHintSpec.java @@ -30,11 +30,10 @@ import java.util.function.Function; /** - * {@link SplitHintSpec} for IngestSegmentFirehoseFactory and DruidInputSource. + * {@link SplitHintSpec} for DruidInputSource. * * In DruidInputSource, this spec is converted into {@link MaxSizeSplitHintSpec}. As a result, its {@link #split} - * method is never called (IngestSegmentFirehoseFactory creates splits on its own instead of calling the - * {@code split()} method). This doesn't necessarily mean this class is deprecated in favor of the MaxSizeSplitHintSpec. + * method is never called. This doesn't necessarily mean this class is deprecated in favor of the MaxSizeSplitHintSpec. * We may want to create more optimized splits in the future. For example, segments can be split to maximize the rollup * ratio if the segments have different sets of columns or even different value ranges of columns. */ diff --git a/processing/src/main/java/org/apache/druid/segment/Metadata.java b/processing/src/main/java/org/apache/druid/segment/Metadata.java index e6b5b1f65883..c3ac554a0faf 100644 --- a/processing/src/main/java/org/apache/druid/segment/Metadata.java +++ b/processing/src/main/java/org/apache/druid/segment/Metadata.java @@ -40,7 +40,7 @@ public class Metadata { // container is used for arbitrary key-value pairs in segment metadata e.g. - // kafka firehose uses it to store commit offset + // kafka input reader uses it to store commit offset private final Map container; @Nullable private final AggregatorFactory[] aggregators; diff --git a/processing/src/test/java/org/apache/druid/data/input/impl/InputEntityIteratingReaderTest.java b/processing/src/test/java/org/apache/druid/data/input/impl/InputEntityIteratingReaderTest.java index 744c29dba2a1..80602d0508ad 100644 --- a/processing/src/test/java/org/apache/druid/data/input/impl/InputEntityIteratingReaderTest.java +++ b/processing/src/test/java/org/apache/druid/data/input/impl/InputEntityIteratingReaderTest.java @@ -187,7 +187,7 @@ public void testSampleWithSystemFields() throws IOException @Test public void testIncorrectURI() throws IOException, URISyntaxException { - final InputEntityIteratingReader firehose = new InputEntityIteratingReader( + final InputEntityIteratingReader inputReader = new InputEntityIteratingReader( new InputRowSchema( new TimestampSpec(null, null, null), new DimensionsSpec( @@ -220,7 +220,7 @@ protected int getMaxRetries() temporaryFolder.newFolder() ); - try (CloseableIterator readIterator = firehose.read()) { + try (CloseableIterator readIterator = inputReader.read()) { String expectedMessage = "Error occurred while trying to read uri: testscheme://some/path"; Exception exception = Assert.assertThrows(RuntimeException.class, readIterator::hasNext); Assert.assertTrue(exception.getMessage().contains(expectedMessage)); diff --git a/server/src/main/java/org/apache/druid/guice/FirehoseModule.java b/server/src/main/java/org/apache/druid/guice/FirehoseModule.java deleted file mode 100644 index fe6461bf2158..000000000000 --- a/server/src/main/java/org/apache/druid/guice/FirehoseModule.java +++ /dev/null @@ -1,55 +0,0 @@ -/* - * Licensed to the Apache Software Foundation (ASF) under one - * or more contributor license agreements. See the NOTICE file - * distributed with this work for additional information - * regarding copyright ownership. The ASF licenses this file - * to you under the Apache License, Version 2.0 (the - * "License"); you may not use this file except in compliance - * with the License. You may obtain a copy of the License at - * - * http://www.apache.org/licenses/LICENSE-2.0 - * - * Unless required by applicable law or agreed to in writing, - * software distributed under the License is distributed on an - * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY - * KIND, either express or implied. See the License for the - * specific language governing permissions and limitations - * under the License. - */ - -package org.apache.druid.guice; - -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 org.apache.druid.initialization.DruidModule; -import org.apache.druid.segment.realtime.firehose.ClippedFirehoseFactory; -import org.apache.druid.segment.realtime.firehose.EventReceiverFirehoseFactory; -import org.apache.druid.segment.realtime.firehose.FixedCountFirehoseFactory; -import org.apache.druid.segment.realtime.firehose.TimedShutoffFirehoseFactory; - -import java.util.Collections; -import java.util.List; - -public class FirehoseModule implements DruidModule -{ - @Override - public void configure(Binder binder) - { - } - - @Override - public List getJacksonModules() - { - return Collections.singletonList( - new SimpleModule("FirehoseModule") - .registerSubtypes( - new NamedType(ClippedFirehoseFactory.class, "clipped"), - new NamedType(TimedShutoffFirehoseFactory.class, "timed"), - new NamedType(EventReceiverFirehoseFactory.class, "receiver"), - new NamedType(FixedCountFirehoseFactory.class, "fixedCount") - ) - ); - } -} diff --git a/server/src/main/java/org/apache/druid/initialization/CoreInjectorBuilder.java b/server/src/main/java/org/apache/druid/initialization/CoreInjectorBuilder.java index 59516f8d4c49..e07ac5ed1155 100644 --- a/server/src/main/java/org/apache/druid/initialization/CoreInjectorBuilder.java +++ b/server/src/main/java/org/apache/druid/initialization/CoreInjectorBuilder.java @@ -29,7 +29,6 @@ import org.apache.druid.guice.DruidSecondaryModule; import org.apache.druid.guice.ExpressionModule; import org.apache.druid.guice.ExtensionsModule; -import org.apache.druid.guice.FirehoseModule; import org.apache.druid.guice.JacksonConfigManagerModule; import org.apache.druid.guice.JavaScriptModule; import org.apache.druid.guice.LifecycleModule; @@ -123,7 +122,6 @@ public CoreInjectorBuilder forServer() new CoordinatorDiscoveryModule(), new LocalDataStorageDruidModule(), new TombstoneDataStorageModule(), - new FirehoseModule(), new JavaScriptModule(), new AuthenticatorModule(), new AuthenticatorMapperModule(), diff --git a/server/src/main/java/org/apache/druid/metadata/BasicDataSourceExt.java b/server/src/main/java/org/apache/druid/metadata/BasicDataSourceExt.java index 6ddfd378fb43..cff9308a6971 100644 --- a/server/src/main/java/org/apache/druid/metadata/BasicDataSourceExt.java +++ b/server/src/main/java/org/apache/druid/metadata/BasicDataSourceExt.java @@ -50,7 +50,7 @@ public class BasicDataSourceExt extends BasicDataSource * Note that these properties are not currently checked against any security configuration such as * an allow list for JDBC properties. Instead, they are supposed to be checked before adding to this class. * - * @see SQLFirehoseDatabaseConnector#validateConfigs + * @see SQLInputSourceDatabaseConnector#validateConfigs */ private Properties connectionProperties; diff --git a/server/src/main/java/org/apache/druid/metadata/SQLFirehoseDatabaseConnector.java b/server/src/main/java/org/apache/druid/metadata/SQLInputSourceDatabaseConnector.java similarity index 98% rename from server/src/main/java/org/apache/druid/metadata/SQLFirehoseDatabaseConnector.java rename to server/src/main/java/org/apache/druid/metadata/SQLInputSourceDatabaseConnector.java index 11d467323f43..02c2d4269c2c 100644 --- a/server/src/main/java/org/apache/druid/metadata/SQLFirehoseDatabaseConnector.java +++ b/server/src/main/java/org/apache/druid/metadata/SQLInputSourceDatabaseConnector.java @@ -38,7 +38,7 @@ import java.util.Set; @JsonTypeInfo(use = JsonTypeInfo.Id.NAME, property = "type") -public abstract class SQLFirehoseDatabaseConnector +public abstract class SQLInputSourceDatabaseConnector { static final int MAX_RETRIES = 10; diff --git a/server/src/main/java/org/apache/druid/metadata/input/SqlEntity.java b/server/src/main/java/org/apache/druid/metadata/input/SqlEntity.java index abc64baae5a6..0d55e0ed7b59 100644 --- a/server/src/main/java/org/apache/druid/metadata/input/SqlEntity.java +++ b/server/src/main/java/org/apache/druid/metadata/input/SqlEntity.java @@ -27,7 +27,7 @@ import org.apache.druid.java.util.common.StringUtils; import org.apache.druid.java.util.common.jackson.JacksonUtils; import org.apache.druid.java.util.common.logger.Logger; -import org.apache.druid.metadata.SQLFirehoseDatabaseConnector; +import org.apache.druid.metadata.SQLInputSourceDatabaseConnector; import org.apache.druid.metadata.SQLMetadataStorageActionHandler; import org.skife.jdbi.v2.ResultIterator; import org.skife.jdbi.v2.exceptions.ResultSetException; @@ -52,19 +52,19 @@ public class SqlEntity implements InputEntity private final String sql; private final ObjectMapper objectMapper; - private final SQLFirehoseDatabaseConnector sqlFirehoseDatabaseConnector; + private final SQLInputSourceDatabaseConnector sqlInputSourceDatabaseConnector; private final boolean foldCase; public SqlEntity( String sql, - SQLFirehoseDatabaseConnector sqlFirehoseDatabaseConnector, + SQLInputSourceDatabaseConnector sqlInputSourceDatabaseConnector, boolean foldCase, ObjectMapper objectMapper ) { this.sql = sql; - this.sqlFirehoseDatabaseConnector = Preconditions.checkNotNull( - sqlFirehoseDatabaseConnector, + this.sqlInputSourceDatabaseConnector = Preconditions.checkNotNull( + sqlInputSourceDatabaseConnector, "SQL Metadata Connector not configured!" ); this.foldCase = foldCase; @@ -93,7 +93,7 @@ public InputStream open() public CleanableFile fetch(File temporaryDirectory, byte[] fetchBuffer) throws IOException { final File tempFile = File.createTempFile("druid-sql-entity", ".tmp", temporaryDirectory); - return openCleanableFile(sql, sqlFirehoseDatabaseConnector, objectMapper, foldCase, tempFile); + return openCleanableFile(sql, sqlInputSourceDatabaseConnector, objectMapper, foldCase, tempFile); } @@ -102,7 +102,7 @@ public CleanableFile fetch(File temporaryDirectory, byte[] fetchBuffer) throws I * The result file is deleted if the query execution or the file write fails. * * @param sql The SQL query to be executed - * @param sqlFirehoseDatabaseConnector The database connector + * @param sqlInputSourceDatabaseConnector The database connector * @param objectMapper An object mapper, used for deserialization * @param foldCase A boolean flag used to enable or disabling case sensitivity while handling database column names * @@ -111,7 +111,7 @@ public CleanableFile fetch(File temporaryDirectory, byte[] fetchBuffer) throws I public static CleanableFile openCleanableFile( String sql, - SQLFirehoseDatabaseConnector sqlFirehoseDatabaseConnector, + SQLInputSourceDatabaseConnector sqlInputSourceDatabaseConnector, ObjectMapper objectMapper, boolean foldCase, File tempFile @@ -124,7 +124,7 @@ public static CleanableFile openCleanableFile( // Execute the sql query and lazily retrieve the results into the file in json format. // foldCase is useful to handle differences in case sensitivity behavior across databases. - sqlFirehoseDatabaseConnector.retryWithHandle( + sqlInputSourceDatabaseConnector.retryWithHandle( (handle) -> { ResultIterator> resultIterator = handle.createQuery( sql @@ -161,7 +161,7 @@ public static CleanableFile openCleanableFile( jg.close(); return null; }, - (exception) -> sqlFirehoseDatabaseConnector.isTransientException(exception) + (exception) -> sqlInputSourceDatabaseConnector.isTransientException(exception) && !(SQLMetadataStorageActionHandler.isStatementException(exception)) ); return new CleanableFile() diff --git a/server/src/main/java/org/apache/druid/metadata/input/SqlInputSource.java b/server/src/main/java/org/apache/druid/metadata/input/SqlInputSource.java index 8d886b058fca..27147babfa5d 100644 --- a/server/src/main/java/org/apache/druid/metadata/input/SqlInputSource.java +++ b/server/src/main/java/org/apache/druid/metadata/input/SqlInputSource.java @@ -36,7 +36,7 @@ import org.apache.druid.data.input.impl.systemfield.SystemFieldDecoratorFactory; import org.apache.druid.guice.annotations.Smile; import org.apache.druid.java.util.common.CloseableIterators; -import org.apache.druid.metadata.SQLFirehoseDatabaseConnector; +import org.apache.druid.metadata.SQLInputSourceDatabaseConnector; import javax.annotation.Nonnull; import javax.annotation.Nullable; @@ -51,7 +51,7 @@ public class SqlInputSource extends AbstractInputSource implements SplittableInp { static final String TYPE_KEY = "sql"; private final List sqls; - private final SQLFirehoseDatabaseConnector sqlFirehoseDatabaseConnector; + private final SQLInputSourceDatabaseConnector sqlInputSourceDatabaseConnector; private final ObjectMapper objectMapper; private final boolean foldCase; @@ -59,7 +59,7 @@ public class SqlInputSource extends AbstractInputSource implements SplittableInp public SqlInputSource( @JsonProperty("sqls") List sqls, @JsonProperty("foldCase") boolean foldCase, - @JsonProperty("database") SQLFirehoseDatabaseConnector sqlFirehoseDatabaseConnector, + @JsonProperty("database") SQLInputSourceDatabaseConnector sqlInputSourceDatabaseConnector, @JacksonInject @Smile ObjectMapper objectMapper ) { @@ -67,8 +67,8 @@ public SqlInputSource( this.sqls = sqls; this.foldCase = foldCase; - this.sqlFirehoseDatabaseConnector = Preconditions.checkNotNull( - sqlFirehoseDatabaseConnector, + this.sqlInputSourceDatabaseConnector = Preconditions.checkNotNull( + sqlInputSourceDatabaseConnector, "SQL Metadata Connector not configured!" ); this.objectMapper = objectMapper; @@ -95,9 +95,9 @@ public boolean isFoldCase() } @JsonProperty("database") - public SQLFirehoseDatabaseConnector getSQLFirehoseDatabaseConnector() + public SQLInputSourceDatabaseConnector getSQLInputSourceDatabaseConnector() { - return sqlFirehoseDatabaseConnector; + return sqlInputSourceDatabaseConnector; } @Override @@ -118,7 +118,7 @@ public SplittableInputSource withSplit(InputSplit split) return new SqlInputSource( Collections.singletonList(split.get()), foldCase, - sqlFirehoseDatabaseConnector, + sqlInputSourceDatabaseConnector, objectMapper ); } @@ -131,7 +131,8 @@ protected InputSourceReader fixedFormatReader(InputRowSchema inputRowSchema, @Nu inputRowSchema, inputFormat, CloseableIterators.withEmptyBaggage(createSplits(inputFormat, null) - .map(split -> new SqlEntity(split.get(), sqlFirehoseDatabaseConnector, foldCase, objectMapper)).iterator()), + .map(split -> new SqlEntity(split.get(), + sqlInputSourceDatabaseConnector, foldCase, objectMapper)).iterator()), SystemFieldDecoratorFactory.NONE, temporaryDirectory ); @@ -155,12 +156,12 @@ public boolean equals(Object o) SqlInputSource that = (SqlInputSource) o; return foldCase == that.foldCase && sqls.equals(that.sqls) && - sqlFirehoseDatabaseConnector.equals(that.sqlFirehoseDatabaseConnector); + sqlInputSourceDatabaseConnector.equals(that.sqlInputSourceDatabaseConnector); } @Override public int hashCode() { - return Objects.hash(sqls, sqlFirehoseDatabaseConnector, foldCase); + return Objects.hash(sqls, sqlInputSourceDatabaseConnector, foldCase); } } diff --git a/server/src/main/java/org/apache/druid/rpc/indexing/SpecificTaskRetryPolicy.java b/server/src/main/java/org/apache/druid/rpc/indexing/SpecificTaskRetryPolicy.java index 36d84f2dc4e7..057201b8564c 100644 --- a/server/src/main/java/org/apache/druid/rpc/indexing/SpecificTaskRetryPolicy.java +++ b/server/src/main/java/org/apache/druid/rpc/indexing/SpecificTaskRetryPolicy.java @@ -23,7 +23,7 @@ import org.apache.druid.java.util.common.StringUtils; import org.apache.druid.rpc.ServiceRetryPolicy; import org.apache.druid.rpc.StandardRetryPolicy; -import org.apache.druid.segment.realtime.firehose.ChatHandlerResource; +import org.apache.druid.segment.realtime.ChatHandlerResource; import org.jboss.netty.handler.codec.http.HttpResponse; import org.jboss.netty.handler.codec.http.HttpResponseStatus; diff --git a/server/src/main/java/org/apache/druid/segment/realtime/firehose/ChatHandler.java b/server/src/main/java/org/apache/druid/segment/realtime/ChatHandler.java similarity index 95% rename from server/src/main/java/org/apache/druid/segment/realtime/firehose/ChatHandler.java rename to server/src/main/java/org/apache/druid/segment/realtime/ChatHandler.java index 004b6c04d480..56f8330fdcbb 100644 --- a/server/src/main/java/org/apache/druid/segment/realtime/firehose/ChatHandler.java +++ b/server/src/main/java/org/apache/druid/segment/realtime/ChatHandler.java @@ -17,7 +17,7 @@ * under the License. */ -package org.apache.druid.segment.realtime.firehose; +package org.apache.druid.segment.realtime; /** * Objects that can be registered with a {@link ServiceAnnouncingChatHandlerProvider} and provide http endpoints for indexing-related diff --git a/server/src/main/java/org/apache/druid/segment/realtime/firehose/ChatHandlerProvider.java b/server/src/main/java/org/apache/druid/segment/realtime/ChatHandlerProvider.java similarity index 97% rename from server/src/main/java/org/apache/druid/segment/realtime/firehose/ChatHandlerProvider.java rename to server/src/main/java/org/apache/druid/segment/realtime/ChatHandlerProvider.java index 3af0d5c37f00..f19e25f37561 100644 --- a/server/src/main/java/org/apache/druid/segment/realtime/firehose/ChatHandlerProvider.java +++ b/server/src/main/java/org/apache/druid/segment/realtime/ChatHandlerProvider.java @@ -17,7 +17,7 @@ * under the License. */ -package org.apache.druid.segment.realtime.firehose; +package org.apache.druid.segment.realtime; import com.google.common.base.Optional; diff --git a/server/src/main/java/org/apache/druid/segment/realtime/firehose/ChatHandlerResource.java b/server/src/main/java/org/apache/druid/segment/realtime/ChatHandlerResource.java similarity index 98% rename from server/src/main/java/org/apache/druid/segment/realtime/firehose/ChatHandlerResource.java rename to server/src/main/java/org/apache/druid/segment/realtime/ChatHandlerResource.java index 1cd579d27340..4423a0d0f0fa 100644 --- a/server/src/main/java/org/apache/druid/segment/realtime/firehose/ChatHandlerResource.java +++ b/server/src/main/java/org/apache/druid/segment/realtime/ChatHandlerResource.java @@ -17,7 +17,7 @@ * under the License. */ -package org.apache.druid.segment.realtime.firehose; +package org.apache.druid.segment.realtime; import com.google.common.base.Optional; import com.google.common.collect.Iterables; diff --git a/server/src/main/java/org/apache/druid/segment/realtime/firehose/ChatHandlers.java b/server/src/main/java/org/apache/druid/segment/realtime/ChatHandlers.java similarity index 97% rename from server/src/main/java/org/apache/druid/segment/realtime/firehose/ChatHandlers.java rename to server/src/main/java/org/apache/druid/segment/realtime/ChatHandlers.java index 4f6e99217d51..4d971db81e08 100644 --- a/server/src/main/java/org/apache/druid/segment/realtime/firehose/ChatHandlers.java +++ b/server/src/main/java/org/apache/druid/segment/realtime/ChatHandlers.java @@ -17,7 +17,7 @@ * under the License. */ -package org.apache.druid.segment.realtime.firehose; +package org.apache.druid.segment.realtime; import org.apache.druid.server.security.Access; import org.apache.druid.server.security.Action; diff --git a/server/src/main/java/org/apache/druid/segment/realtime/FireHydrant.java b/server/src/main/java/org/apache/druid/segment/realtime/FireHydrant.java index 5f1a88f2ea97..b051fa72724d 100644 --- a/server/src/main/java/org/apache/druid/segment/realtime/FireHydrant.java +++ b/server/src/main/java/org/apache/druid/segment/realtime/FireHydrant.java @@ -44,6 +44,7 @@ public class FireHydrant { private final int count; private final AtomicReference adapter; + @Nullable private volatile IncrementalIndex index; public FireHydrant(IncrementalIndex index, int count, SegmentId segmentId) @@ -62,6 +63,7 @@ public FireHydrant(Segment adapter, int count) this.count = count; } + @Nullable public IncrementalIndex getIndex() { return index; diff --git a/server/src/main/java/org/apache/druid/segment/realtime/firehose/NoopChatHandlerProvider.java b/server/src/main/java/org/apache/druid/segment/realtime/NoopChatHandlerProvider.java similarity index 96% rename from server/src/main/java/org/apache/druid/segment/realtime/firehose/NoopChatHandlerProvider.java rename to server/src/main/java/org/apache/druid/segment/realtime/NoopChatHandlerProvider.java index 5c49b9ac292e..9480bdf2bc7e 100644 --- a/server/src/main/java/org/apache/druid/segment/realtime/firehose/NoopChatHandlerProvider.java +++ b/server/src/main/java/org/apache/druid/segment/realtime/NoopChatHandlerProvider.java @@ -17,7 +17,7 @@ * under the License. */ -package org.apache.druid.segment.realtime.firehose; +package org.apache.druid.segment.realtime; import com.google.common.base.Optional; diff --git a/server/src/main/java/org/apache/druid/segment/realtime/firehose/ServiceAnnouncingChatHandlerProvider.java b/server/src/main/java/org/apache/druid/segment/realtime/ServiceAnnouncingChatHandlerProvider.java similarity index 98% rename from server/src/main/java/org/apache/druid/segment/realtime/firehose/ServiceAnnouncingChatHandlerProvider.java rename to server/src/main/java/org/apache/druid/segment/realtime/ServiceAnnouncingChatHandlerProvider.java index 802be54cc115..da3975e4545a 100644 --- a/server/src/main/java/org/apache/druid/segment/realtime/firehose/ServiceAnnouncingChatHandlerProvider.java +++ b/server/src/main/java/org/apache/druid/segment/realtime/ServiceAnnouncingChatHandlerProvider.java @@ -17,7 +17,7 @@ * under the License. */ -package org.apache.druid.segment.realtime.firehose; +package org.apache.druid.segment.realtime; import com.google.common.base.Optional; import com.google.inject.Inject; diff --git a/server/src/main/java/org/apache/druid/segment/realtime/firehose/WindowedStorageAdapter.java b/server/src/main/java/org/apache/druid/segment/realtime/WindowedStorageAdapter.java similarity index 96% rename from server/src/main/java/org/apache/druid/segment/realtime/firehose/WindowedStorageAdapter.java rename to server/src/main/java/org/apache/druid/segment/realtime/WindowedStorageAdapter.java index 26d0b41013da..9b4bd1e4ea7f 100644 --- a/server/src/main/java/org/apache/druid/segment/realtime/firehose/WindowedStorageAdapter.java +++ b/server/src/main/java/org/apache/druid/segment/realtime/WindowedStorageAdapter.java @@ -17,7 +17,7 @@ * under the License. */ -package org.apache.druid.segment.realtime.firehose; +package org.apache.druid.segment.realtime; import org.apache.druid.segment.StorageAdapter; import org.joda.time.Interval; diff --git a/server/src/main/java/org/apache/druid/segment/realtime/appenderator/AppenderatorImpl.java b/server/src/main/java/org/apache/druid/segment/realtime/appenderator/AppenderatorImpl.java index 734abc6ed542..d9911366bdf3 100644 --- a/server/src/main/java/org/apache/druid/segment/realtime/appenderator/AppenderatorImpl.java +++ b/server/src/main/java/org/apache/druid/segment/realtime/appenderator/AppenderatorImpl.java @@ -1177,7 +1177,7 @@ private void initializeExecutors() final int maxPendingPersists = tuningConfig.getMaxPendingPersists(); if (persistExecutor == null) { - // use a blocking single threaded executor to throttle the firehose when write to disk is slow + // use a blocking single threaded executor to throttle the input source when write to disk is slow persistExecutor = MoreExecutors.listeningDecorator( Execs.newBlockingThreaded( "[" + StringUtils.encodeForFormat(myId) + "]-appenderator-persist", @@ -1187,7 +1187,7 @@ private void initializeExecutors() } if (pushExecutor == null) { - // use a blocking single threaded executor to throttle the firehose when write to disk is slow + // use a blocking single threaded executor to throttle the input source when write to disk is slow pushExecutor = MoreExecutors.listeningDecorator( Execs.newBlockingSingleThreaded("[" + StringUtils.encodeForFormat(myId) + "]-appenderator-merge", 1) ); diff --git a/server/src/main/java/org/apache/druid/segment/realtime/appenderator/BatchAppenderator.java b/server/src/main/java/org/apache/druid/segment/realtime/appenderator/BatchAppenderator.java index b544d33705cc..979d8cb92b3d 100644 --- a/server/src/main/java/org/apache/druid/segment/realtime/appenderator/BatchAppenderator.java +++ b/server/src/main/java/org/apache/druid/segment/realtime/appenderator/BatchAppenderator.java @@ -236,7 +236,7 @@ private void initializeExecutors() } if (pushExecutor == null) { - // use a blocking single threaded executor to throttle the firehose when write to disk is slow + // use a blocking single threaded executor to throttle the input source when write to disk is slow pushExecutor = MoreExecutors.listeningDecorator( Execs.newBlockingSingleThreaded( "[" + StringUtils.encodeForFormat(myId) + "]-batch-appenderator-push", diff --git a/server/src/main/java/org/apache/druid/segment/realtime/appenderator/StreamAppenderator.java b/server/src/main/java/org/apache/druid/segment/realtime/appenderator/StreamAppenderator.java index 4d1253591e08..a25d6b7acd77 100644 --- a/server/src/main/java/org/apache/druid/segment/realtime/appenderator/StreamAppenderator.java +++ b/server/src/main/java/org/apache/druid/segment/realtime/appenderator/StreamAppenderator.java @@ -1234,7 +1234,7 @@ private void initializeExecutors() } if (pushExecutor == null) { - // use a blocking single threaded executor to throttle the firehose when write to disk is slow + // use a blocking single threaded executor to throttle the input source when write to disk is slow pushExecutor = MoreExecutors.listeningDecorator( Execs.newBlockingSingleThreaded("[" + StringUtils.encodeForFormat(myId) + "]-appenderator-merge", 1) ); diff --git a/server/src/main/java/org/apache/druid/segment/realtime/firehose/ClippedFirehoseFactory.java b/server/src/main/java/org/apache/druid/segment/realtime/firehose/ClippedFirehoseFactory.java deleted file mode 100644 index 9ca595ce5c28..000000000000 --- a/server/src/main/java/org/apache/druid/segment/realtime/firehose/ClippedFirehoseFactory.java +++ /dev/null @@ -1,81 +0,0 @@ -/* - * Licensed to the Apache Software Foundation (ASF) under one - * or more contributor license agreements. See the NOTICE file - * distributed with this work for additional information - * regarding copyright ownership. The ASF licenses this file - * to you under the Apache License, Version 2.0 (the - * "License"); you may not use this file except in compliance - * with the License. You may obtain a copy of the License at - * - * http://www.apache.org/licenses/LICENSE-2.0 - * - * Unless required by applicable law or agreed to in writing, - * software distributed under the License is distributed on an - * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY - * KIND, either express or implied. See the License for the - * specific language governing permissions and limitations - * under the License. - */ - -package org.apache.druid.segment.realtime.firehose; - -import com.fasterxml.jackson.annotation.JsonCreator; -import com.fasterxml.jackson.annotation.JsonProperty; -import com.google.common.base.Predicate; -import org.apache.druid.data.input.Firehose; -import org.apache.druid.data.input.FirehoseFactory; -import org.apache.druid.data.input.InputRow; -import org.apache.druid.data.input.impl.InputRowParser; -import org.joda.time.Interval; - -import java.io.File; -import java.io.IOException; - -/** - * Creates firehoses clipped to a particular time interval. Useful for enforcing min time, max time, and time windows. - */ -@Deprecated -public class ClippedFirehoseFactory implements FirehoseFactory -{ - private final FirehoseFactory delegate; - private final Interval interval; - - @JsonCreator - public ClippedFirehoseFactory( - @JsonProperty("delegate") FirehoseFactory delegate, - @JsonProperty("interval") Interval interval - ) - { - this.delegate = delegate; - this.interval = interval; - } - - @JsonProperty - public FirehoseFactory getDelegate() - { - return delegate; - } - - @JsonProperty - public Interval getInterval() - { - return interval; - } - - @Override - public Firehose connect(InputRowParser parser, File temporaryDirectory) throws IOException - { - return new PredicateFirehose( - delegate.connect(parser, temporaryDirectory), - new Predicate() - { - @Override - public boolean apply(InputRow input) - { - return interval.contains(input.getTimestampFromEpoch()); - } - } - ); - } - -} diff --git a/server/src/main/java/org/apache/druid/segment/realtime/firehose/EventReceiver.java b/server/src/main/java/org/apache/druid/segment/realtime/firehose/EventReceiver.java deleted file mode 100644 index e2b776fa4fa5..000000000000 --- a/server/src/main/java/org/apache/druid/segment/realtime/firehose/EventReceiver.java +++ /dev/null @@ -1,28 +0,0 @@ -/* - * Licensed to the Apache Software Foundation (ASF) under one - * or more contributor license agreements. See the NOTICE file - * distributed with this work for additional information - * regarding copyright ownership. The ASF licenses this file - * to you under the Apache License, Version 2.0 (the - * "License"); you may not use this file except in compliance - * with the License. You may obtain a copy of the License at - * - * http://www.apache.org/licenses/LICENSE-2.0 - * - * Unless required by applicable law or agreed to in writing, - * software distributed under the License is distributed on an - * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY - * KIND, either express or implied. See the License for the - * specific language governing permissions and limitations - * under the License. - */ - -package org.apache.druid.segment.realtime.firehose; - -import java.util.Collection; -import java.util.Map; - -public interface EventReceiver -{ - void addAll(Collection> events); -} diff --git a/server/src/main/java/org/apache/druid/segment/realtime/firehose/EventReceiverFirehoseFactory.java b/server/src/main/java/org/apache/druid/segment/realtime/firehose/EventReceiverFirehoseFactory.java deleted file mode 100644 index 4fad62a6d462..000000000000 --- a/server/src/main/java/org/apache/druid/segment/realtime/firehose/EventReceiverFirehoseFactory.java +++ /dev/null @@ -1,659 +0,0 @@ -/* - * Licensed to the Apache Software Foundation (ASF) under one - * or more contributor license agreements. See the NOTICE file - * distributed with this work for additional information - * regarding copyright ownership. The ASF licenses this file - * to you under the Apache License, Version 2.0 (the - * "License"); you may not use this file except in compliance - * with the License. You may obtain a copy of the License at - * - * http://www.apache.org/licenses/LICENSE-2.0 - * - * Unless required by applicable law or agreed to in writing, - * software distributed under the License is distributed on an - * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY - * KIND, either express or implied. See the License for the - * specific language governing permissions and limitations - * under the License. - */ - -package org.apache.druid.segment.realtime.firehose; - -import com.fasterxml.jackson.annotation.JacksonInject; -import com.fasterxml.jackson.annotation.JsonCreator; -import com.fasterxml.jackson.annotation.JsonProperty; -import com.fasterxml.jackson.core.JsonProcessingException; -import com.fasterxml.jackson.core.type.TypeReference; -import com.fasterxml.jackson.databind.ObjectMapper; -import com.fasterxml.jackson.jaxrs.smile.SmileMediaTypes; -import com.google.common.annotations.VisibleForTesting; -import com.google.common.base.Preconditions; -import com.google.common.collect.ImmutableMap; -import com.google.common.io.CountingInputStream; -import com.google.common.util.concurrent.Uninterruptibles; -import com.google.errorprone.annotations.concurrent.GuardedBy; -import org.apache.druid.concurrent.Threads; -import org.apache.druid.data.input.Firehose; -import org.apache.druid.data.input.FirehoseFactory; -import org.apache.druid.data.input.InputRow; -import org.apache.druid.data.input.impl.InputRowParser; -import org.apache.druid.guice.annotations.Json; -import org.apache.druid.guice.annotations.Smile; -import org.apache.druid.java.util.common.DateTimes; -import org.apache.druid.java.util.emitter.EmittingLogger; -import org.apache.druid.server.metrics.EventReceiverFirehoseMetric; -import org.apache.druid.server.metrics.EventReceiverFirehoseRegister; -import org.apache.druid.server.security.Access; -import org.apache.druid.server.security.Action; -import org.apache.druid.server.security.AuthorizationUtils; -import org.apache.druid.server.security.AuthorizerMapper; -import org.apache.druid.server.security.Resource; -import org.apache.druid.server.security.ResourceAction; -import org.joda.time.DateTime; - -import javax.annotation.Nullable; -import javax.servlet.http.HttpServletRequest; -import javax.ws.rs.Consumes; -import javax.ws.rs.POST; -import javax.ws.rs.Path; -import javax.ws.rs.Produces; -import javax.ws.rs.QueryParam; -import javax.ws.rs.core.Context; -import javax.ws.rs.core.MediaType; -import javax.ws.rs.core.Response; -import java.io.File; -import java.io.IOException; -import java.io.InputStream; -import java.util.ArrayList; -import java.util.Collection; -import java.util.List; -import java.util.Map; -import java.util.NoSuchElementException; -import java.util.concurrent.ArrayBlockingQueue; -import java.util.concurrent.BlockingQueue; -import java.util.concurrent.ConcurrentHashMap; -import java.util.concurrent.TimeUnit; -import java.util.concurrent.atomic.AtomicLong; - -/** - * Builds firehoses that accept events through the {@link EventReceiver} interface. Can also register these - * firehoses with an {@link ServiceAnnouncingChatHandlerProvider}. - */ -@Deprecated -public class EventReceiverFirehoseFactory implements FirehoseFactory>> -{ - private static final EmittingLogger log = new EmittingLogger(EventReceiverFirehoseFactory.class); - - public static final int MAX_FIREHOSE_PRODUCERS = 10_000; - - private static final int DEFAULT_BUFFER_SIZE = 100_000; - - /** - * A "poison pill" object for {@link EventReceiverFirehose}'s internal buffer. - */ - private static final Object FIREHOSE_CLOSED = new Object(); - - private final String serviceName; - private final int bufferSize; - - /** - * Doesn't really support max idle times finer than 1 second due to how {@link - * EventReceiverFirehose#delayedCloseExecutor} is implemented, see a comment inside {@link - * EventReceiverFirehose#createDelayedCloseExecutor()}. This aspect is not reflected in docs because it's unlikely - * that anybody configures or cares about finer max idle times, and also because this is an implementation detail of - * {@link EventReceiverFirehose} that may change in the future. - */ - private final long maxIdleTimeMillis; - private final ChatHandlerProvider chatHandlerProvider; - private final ObjectMapper jsonMapper; - private final ObjectMapper smileMapper; - private final EventReceiverFirehoseRegister eventReceiverFirehoseRegister; - private final AuthorizerMapper authorizerMapper; - - @JsonCreator - public EventReceiverFirehoseFactory( - @JsonProperty("serviceName") String serviceName, - @JsonProperty("bufferSize") Integer bufferSize, - // Keeping the legacy 'maxIdleTime' property name for backward compatibility. When the project is updated to - // Jackson 2.9 it could be changed, see https://github.com/apache/druid/issues/7152 - @JsonProperty("maxIdleTime") @Nullable Long maxIdleTimeMillis, - @JacksonInject ChatHandlerProvider chatHandlerProvider, - @JacksonInject @Json ObjectMapper jsonMapper, - @JacksonInject @Smile ObjectMapper smileMapper, - @JacksonInject EventReceiverFirehoseRegister eventReceiverFirehoseRegister, - @JacksonInject AuthorizerMapper authorizerMapper - ) - { - Preconditions.checkNotNull(serviceName, "serviceName"); - - this.serviceName = serviceName; - this.bufferSize = bufferSize == null || bufferSize <= 0 ? DEFAULT_BUFFER_SIZE : bufferSize; - this.maxIdleTimeMillis = (maxIdleTimeMillis == null || maxIdleTimeMillis <= 0) ? Long.MAX_VALUE : maxIdleTimeMillis; - this.chatHandlerProvider = chatHandlerProvider; - this.jsonMapper = jsonMapper; - this.smileMapper = smileMapper; - this.eventReceiverFirehoseRegister = eventReceiverFirehoseRegister; - this.authorizerMapper = authorizerMapper; - } - - @Override - public Firehose connect( - InputRowParser> firehoseParser, - File temporaryDirectory - ) - { - log.info("Connecting firehose: %s", serviceName); - final EventReceiverFirehose firehose = new EventReceiverFirehose(firehoseParser); - - if (chatHandlerProvider != null) { - log.info("Found chathandler of class[%s]", chatHandlerProvider.getClass().getName()); - chatHandlerProvider.register(serviceName, firehose); - int lastIndexOfColon = serviceName.lastIndexOf(':'); - if (lastIndexOfColon > 0) { - chatHandlerProvider.register(serviceName.substring(lastIndexOfColon + 1), firehose); - } - } else { - log.warn("No chathandler detected"); - } - - eventReceiverFirehoseRegister.register(serviceName, firehose); - - return firehose; - } - - @JsonProperty - public String getServiceName() - { - return serviceName; - } - - @JsonProperty - public int getBufferSize() - { - return bufferSize; - } - - /** - * Keeping the legacy 'maxIdleTime' property name for backward compatibility. When the project is updated to Jackson - * 2.9 it could be changed, see https://github.com/apache/druid/issues/7152 - */ - @JsonProperty("maxIdleTime") - public long getMaxIdleTimeMillis() - { - return maxIdleTimeMillis; - } - - /** - * Apart from adhering to {@link Firehose} contract regarding concurrency, this class has two methods that might be - * called concurrently with any other methods and each other, from arbitrary number of threads: {@link #addAll} and - * {@link #shutdown}. - * - * Concurrent data flow: in {@link #addAll} (can be called concurrently with any other methods and other calls to - * {@link #addAll}) rows are pushed into {@link #buffer}. The single Firehose "consumer" thread calls {@link #hasMore} - * and {@link #nextRow()}, where rows are taken out from the other end of the {@link #buffer} queue. - * - * This class creates and manages one thread ({@link #delayedCloseExecutor}) for calling {@link #close()} - * asynchronously in response to a {@link #shutdown} request, or after this Firehose has been idle (no calls to {@link - * #addAll}) for {@link #maxIdleTimeMillis}. - */ - @VisibleForTesting - public class EventReceiverFirehose implements ChatHandler, Firehose, EventReceiverFirehoseMetric - { - /** - * How does this thread work (and its interruption policy) is described in the comment for {@link - * #createDelayedCloseExecutor}. - */ - @GuardedBy("this") - private @Nullable Thread delayedCloseExecutor; - - /** - * Contains {@link InputRow} objects, the last one is {@link #FIREHOSE_CLOSED} which is a "poison pill". Poison pill - * is used to notify the thread that calls {@link #hasMore()} and {@link #nextRow()} that the EventReceiverFirehose - * is closed without heuristic 500 ms timed blocking in a loop instead of a simple {@link BlockingQueue#take()} - * call (see {@link #hasMore} code). - */ - private final BlockingQueue buffer; - private final InputRowParser> parser; - - /** - * This field needs to be volatile to ensure progress in {@link #addRows} method where it is read in a loop, and - * also in testing code calling {@link #isClosed()}. - */ - private volatile boolean closed = false; - - /** - * This field and {@link #rowsRunOut} are not volatile because they are accessed only from {@link #hasMore()} and - * {@link #nextRow()} methods that are called from a single thread according to {@link Firehose} spec. - */ - @Nullable - private InputRow nextRow = null; - private boolean rowsRunOut = false; - - private final AtomicLong bytesReceived = new AtomicLong(0); - private final AtomicLong lastBufferAddFailLoggingTimeNs = new AtomicLong(System.nanoTime()); - private final ConcurrentHashMap producerSequences = new ConcurrentHashMap<>(); - - /** - * This field and {@link #requestedShutdownTimeNs} use nanoseconds instead of milliseconds not to deal with the fact - * that {@link System#currentTimeMillis()} can "go backward", e. g. due to time correction on the server. - * - * This field and {@link #requestedShutdownTimeNs} must be volatile because they are de facto lazily initialized - * fields that are used concurrently in {@link #delayedCloseExecutor} (see {@link #createDelayedCloseExecutor()}). - * If they were not volatile, NPE would be possible in {@link #delayedCloseExecutor}. See - * https://shipilev.net/blog/2016/close-encounters-of-jmm-kind/#wishful-hb-actual for explanations. - */ - @Nullable - private volatile Long idleCloseTimeNs = null; - @Nullable - private volatile Long requestedShutdownTimeNs = null; - - EventReceiverFirehose(InputRowParser> parser) - { - this.buffer = new ArrayBlockingQueue<>(bufferSize); - this.parser = parser; - - if (maxIdleTimeMillis != Long.MAX_VALUE) { - idleCloseTimeNs = System.nanoTime() + TimeUnit.MILLISECONDS.toNanos(maxIdleTimeMillis); - synchronized (this) { - createDelayedCloseExecutor(); - } - } - } - - @VisibleForTesting - synchronized @Nullable Thread getDelayedCloseExecutor() - { - return delayedCloseExecutor; - } - - /** - * Creates and starts a {@link #delayedCloseExecutor} thread, either right from the EventReceiverFirehose's - * constructor if {@link #maxIdleTimeMillis} is specified, or otherwise lazily from {@link #shutdown}. - * - * The thread waits until the time when the Firehose should be closed because either {@link #addAll} was not called - * for the specified max idle time (see {@link #idleCloseTimeNs}), or until the shutoff time requested last - * via {@link #shutdown} (see {@link #requestedShutdownTimeNs}), whatever is sooner. Then the thread does - * two things: - * 1. if the Firehose is already closed (or in the process of closing, but {@link #closed} flag is already set), it - * silently exits. - * 2. It checks both deadlines again: - * a) if either of them has arrived, it calls {@link #close()} and exits. - * b) otherwise, it waits until the nearest deadline again, and so on in a loop. - * - * This way the thread works predictably and robustly regardless of how both deadlines change (for example, shutoff - * time specified via {@link #shutdown} may jump in both directions). - * - * Other methods notify {@link #delayedCloseExecutor} that the Firehose state in some way that is important for this - * thread (that is, when {@link #close()} is called, {@link #delayedCloseExecutor} is no longer needed and should - * exit as soon as possible to release system resources; when {@link #shutdown} is called, the thread may need to - * wake up sooner if the shutoff time has been moved sooner) by simply interrupting it. The thread wakes up and - * continues its loop. - */ - @GuardedBy("this") - private Thread createDelayedCloseExecutor() - { - Thread delayedCloseExecutor = new Thread( - () -> { - // The closed = true is visible after close() because there is a happens-before edge between - // delayedCloseExecutor.interrupt() call in close() and catching InterruptedException below in this loop. - while (!closed) { - if (idleCloseTimeNs == null && requestedShutdownTimeNs == null) { - // This is not possible unless there are bugs in the code of EventReceiverFirehose. AssertionError could - // have been thrown instead, but it doesn't seem to make a lot of sense in a background thread. Instead, - // we long the error and continue a loop after some pause. - log.error( - "Either idleCloseTimeNs or requestedShutdownTimeNs must be non-null. " - + "Please file a bug at https://github.com/apache/druid/issues" - ); - } - if (idleCloseTimeNs != null && idleCloseTimeNs - System.nanoTime() <= 0) { // overflow-aware comparison - log.info("Firehose has been idle for %d ms, closing.", maxIdleTimeMillis); - close(); - } else if (requestedShutdownTimeNs != null && - requestedShutdownTimeNs - System.nanoTime() <= 0) { // overflow-aware comparison - log.info("Closing Firehose after a shutdown request"); - close(); - } - try { - // It is possible to write code that sleeps until the next the next idleCloseTimeNs or - // requestedShutdownTimeNs, whatever is non-null and sooner, but that's fairly complicated code. That - // complexity perhaps overweighs the minor inefficiency of simply waking up every second. - Threads.sleepFor(1, TimeUnit.SECONDS); - } - catch (InterruptedException ignore) { - // Interruption is a wakeup, continue the loop - } - } - }, - "event-receiver-firehose-closer" - ); - delayedCloseExecutor.setDaemon(true); - this.delayedCloseExecutor = delayedCloseExecutor; - delayedCloseExecutor.start(); - return delayedCloseExecutor; - } - - /** - * This method might be called concurrently from multiple threads, if multiple requests arrive to the server at the - * same time (possibly exact duplicates). Concurrency is controlled in {@link #checkProducerSequence}, where only - * requests with "X-Firehose-Producer-Seq" number greater than the max "X-Firehose-Producer-Seq" in previously - * arrived requests are allowed to proceed. After that check requests don't synchronize with each other and - * therefore if two large batches are sent with little interval, the events from the batches might be mixed up in - * {@link #buffer} (if two {@link #addRows(Iterable)} are executed concurrently). - */ - @POST - @Path("/push-events") - @Consumes({MediaType.APPLICATION_JSON, SmileMediaTypes.APPLICATION_JACKSON_SMILE}) - @Produces({MediaType.APPLICATION_JSON, SmileMediaTypes.APPLICATION_JACKSON_SMILE}) - public Response addAll(InputStream in, @Context final HttpServletRequest req) throws JsonProcessingException - { - idleCloseTimeNs = System.nanoTime() + TimeUnit.MILLISECONDS.toNanos(maxIdleTimeMillis); - Access accessResult = AuthorizationUtils.authorizeResourceAction( - req, - new ResourceAction( - Resource.STATE_RESOURCE, - Action.WRITE - ), - authorizerMapper - ); - if (!accessResult.isAllowed()) { - return Response.status(403).build(); - } - - final String reqContentType = req.getContentType(); - final boolean isSmile = SmileMediaTypes.APPLICATION_JACKSON_SMILE.equals(reqContentType); - final String contentType = isSmile ? SmileMediaTypes.APPLICATION_JACKSON_SMILE : MediaType.APPLICATION_JSON; - - ObjectMapper objectMapper = isSmile ? smileMapper : jsonMapper; - - Response producerSequenceResponse = checkProducerSequence(req, reqContentType, objectMapper); - if (producerSequenceResponse != null) { - return producerSequenceResponse; - } - - CountingInputStream countingInputStream = new CountingInputStream(in); - Collection> events; - try { - events = objectMapper.readValue( - countingInputStream, - new TypeReference>>() - { - } - ); - } - catch (IOException e) { - return Response.serverError().entity(ImmutableMap.of("error", e.getMessage())).build(); - } - finally { - bytesReceived.addAndGet(countingInputStream.getCount()); - } - log.debug("Adding %,d events to firehose: %s", events.size(), serviceName); - - final List rows = new ArrayList<>(); - for (final Map event : events) { - // Might throw an exception. We'd like that to happen now, instead of while adding to the row buffer. - rows.addAll(parser.parseBatch(event)); - } - - try { - addRows(rows); - return Response.ok( - objectMapper.writeValueAsString(ImmutableMap.of("eventCount", events.size())), - contentType - ).build(); - } - catch (InterruptedException e) { - Thread.currentThread().interrupt(); - throw new RuntimeException(e); - } - } - - @Override - public boolean hasMore() - { - if (rowsRunOut) { - return false; - } - if (nextRow != null) { - return true; - } - Object next; - try { - next = buffer.take(); - } - catch (InterruptedException e) { - Thread.currentThread().interrupt(); - throw new RuntimeException(e); - } - //noinspection ObjectEquality - if (next == FIREHOSE_CLOSED) { - rowsRunOut = true; - return false; - } - nextRow = (InputRow) next; - return true; - } - - @Nullable - @Override - public InputRow nextRow() - { - final InputRow row = nextRow; - - if (row == null) { - throw new NoSuchElementException(); - } else { - nextRow = null; - return row; - } - } - - @Override - public int getCurrentBufferSize() - { - return buffer.size(); - } - - @Override - public int getCapacity() - { - return bufferSize; - } - - @Override - public long getBytesReceived() - { - return bytesReceived.get(); - } - - /** - * This method is synchronized because it might be called concurrently from multiple threads: from {@link - * #delayedCloseExecutor}, and from the thread that creates and uses the Firehose object. - */ - @Override - public synchronized void close() - { - if (closed) { - return; - } - closed = true; - log.info("Firehose closing."); - - // Critical to add the poison pill to the queue, don't allow interruption. - Uninterruptibles.putUninterruptibly(buffer, FIREHOSE_CLOSED); - - eventReceiverFirehoseRegister.unregister(serviceName); - if (chatHandlerProvider != null) { - chatHandlerProvider.unregister(serviceName); - } - if (delayedCloseExecutor != null && !delayedCloseExecutor.equals(Thread.currentThread())) { - // Interrupt delayedCloseExecutor to let it discover that closed flag is already set and exit. - delayedCloseExecutor.interrupt(); - } - } - - @VisibleForTesting - void addRows(Iterable rows) throws InterruptedException - { - for (final InputRow row : rows) { - boolean added = false; - while (!closed && !added) { - added = buffer.offer(row, 500, TimeUnit.MILLISECONDS); - if (!added) { - long currTimeNs = System.nanoTime(); - long lastTimeNs = lastBufferAddFailLoggingTimeNs.get(); - if (currTimeNs - lastTimeNs > TimeUnit.SECONDS.toNanos(10) && - lastBufferAddFailLoggingTimeNs.compareAndSet(lastTimeNs, currTimeNs)) { - log.warn("Failed to add event to buffer with current size [%s] . Retrying...", buffer.size()); - } - } - } - - if (!added) { - throw new IllegalStateException("Cannot add events to closed firehose!"); - } - } - } - - /** - * This method might be called concurrently from multiple threads, if multiple shutdown requests arrive at the same - * time. No attempts are made to synchronize such requests, or prioritize them a-la "latest shutdown time wins" or - * "soonest shutdown time wins". {@link #delayedCloseExecutor}'s logic (see {@link #createDelayedCloseExecutor()}) - * is indifferent to shutdown times jumping in arbitrary directions. But once a shutdown request is made, it can't - * be cancelled entirely, the shutdown time could only be rescheduled with a new request. - */ - @POST - @Path("/shutdown") - @Consumes({MediaType.APPLICATION_JSON, SmileMediaTypes.APPLICATION_JACKSON_SMILE}) - @Produces({MediaType.APPLICATION_JSON, SmileMediaTypes.APPLICATION_JACKSON_SMILE}) - public Response shutdown( - @QueryParam("shutoffTime") final String shutoffTimeMillis, - @Context final HttpServletRequest req - ) - { - Access accessResult = AuthorizationUtils.authorizeResourceAction( - req, - new ResourceAction( - Resource.STATE_RESOURCE, - Action.WRITE - ), - authorizerMapper - ); - if (!accessResult.isAllowed()) { - return Response.status(403).build(); - } - - try { - DateTime shutoffAt = shutoffTimeMillis == null ? DateTimes.nowUtc() : DateTimes.of(shutoffTimeMillis); - log.info("Setting Firehose shutoffTime to %s", shutoffTimeMillis); - long shutoffTimeoutMillis = Math.max(shutoffAt.getMillis() - System.currentTimeMillis(), 0); - - requestedShutdownTimeNs = System.nanoTime() + TimeUnit.MILLISECONDS.toNanos(shutoffTimeoutMillis); - Thread delayedCloseExecutor; - // Need to interrupt delayedCloseExecutor because a newly specified shutdown time might be closer than idle - // timeout or previously specified shutdown. Interruption of delayedCloseExecutor lets it adjust the sleep time - // (see the logic of this thread in createDelayedCloseExecutor()). - boolean needToInterruptDelayedCloseExecutor = true; - synchronized (this) { - delayedCloseExecutor = this.delayedCloseExecutor; - if (delayedCloseExecutor == null) { - delayedCloseExecutor = createDelayedCloseExecutor(); - // Don't need to interrupt a freshly created thread - needToInterruptDelayedCloseExecutor = false; - } - } - if (needToInterruptDelayedCloseExecutor) { - delayedCloseExecutor.interrupt(); - } - return Response.ok().build(); - } - catch (IllegalArgumentException e) { - return Response.status(Response.Status.BAD_REQUEST) - .entity(ImmutableMap.of("error", e.getMessage())) - .build(); - - } - } - - @VisibleForTesting - boolean isClosed() - { - return closed; - } - - /** - * Checks the request for a producer ID and sequence value. If the producer ID is specified, a corresponding - * sequence value must be specified as well. If the incoming sequence is less than or equal to the last seen - * sequence for that producer ID, the request is ignored. - * - * This method might be called concurrently from multiple threads. - * - * @param req Http request - * @param responseContentType Response content type - * @param responseMapper Response object mapper - * @return an error response to return or null if the request can proceed - */ - @Nullable - private Response checkProducerSequence( - final HttpServletRequest req, - final String responseContentType, - final ObjectMapper responseMapper - ) - { - final String producerId = req.getHeader("X-Firehose-Producer-Id"); - - if (producerId == null) { - return null; - } - - final String sequenceValue = req.getHeader("X-Firehose-Producer-Seq"); - - if (sequenceValue == null) { - return Response - .status(Response.Status.BAD_REQUEST) - .entity(ImmutableMap.of("error", "Producer sequence value is missing")) - .build(); - } - - Long producerSequence = producerSequences.computeIfAbsent(producerId, key -> Long.MIN_VALUE); - - if (producerSequences.size() >= MAX_FIREHOSE_PRODUCERS) { - return Response - .status(Response.Status.FORBIDDEN) - .entity( - ImmutableMap.of( - "error", - "Too many individual producer IDs for this firehose. Max is " + MAX_FIREHOSE_PRODUCERS - ) - ) - .build(); - } - - try { - Long newSequence = Long.parseLong(sequenceValue); - - while (true) { - if (newSequence <= producerSequence) { - return Response.ok( - responseMapper.writeValueAsString(ImmutableMap.of("eventCount", 0, "skipped", true)), - responseContentType - ).build(); - } - if (producerSequences.replace(producerId, producerSequence, newSequence)) { - return null; - } - producerSequence = producerSequences.get(producerId); - } - } - catch (JsonProcessingException ex) { - throw new RuntimeException(ex); - } - catch (NumberFormatException ex) { - return Response - .status(Response.Status.BAD_REQUEST) - .entity(ImmutableMap.of("error", "Producer sequence must be a number")) - .build(); - } - } - } -} diff --git a/server/src/main/java/org/apache/druid/segment/realtime/firehose/FixedCountFirehoseFactory.java b/server/src/main/java/org/apache/druid/segment/realtime/firehose/FixedCountFirehoseFactory.java deleted file mode 100644 index 72a48b33a45a..000000000000 --- a/server/src/main/java/org/apache/druid/segment/realtime/firehose/FixedCountFirehoseFactory.java +++ /dev/null @@ -1,93 +0,0 @@ -/* - * Licensed to the Apache Software Foundation (ASF) under one - * or more contributor license agreements. See the NOTICE file - * distributed with this work for additional information - * regarding copyright ownership. The ASF licenses this file - * to you under the Apache License, Version 2.0 (the - * "License"); you may not use this file except in compliance - * with the License. You may obtain a copy of the License at - * - * http://www.apache.org/licenses/LICENSE-2.0 - * - * Unless required by applicable law or agreed to in writing, - * software distributed under the License is distributed on an - * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY - * KIND, either express or implied. See the License for the - * specific language governing permissions and limitations - * under the License. - */ - -package org.apache.druid.segment.realtime.firehose; - -import com.fasterxml.jackson.annotation.JsonCreator; -import com.fasterxml.jackson.annotation.JsonProperty; -import com.google.common.base.Preconditions; -import org.apache.druid.data.input.Firehose; -import org.apache.druid.data.input.FirehoseFactory; -import org.apache.druid.data.input.InputRow; -import org.apache.druid.data.input.impl.InputRowParser; - -import javax.annotation.Nullable; -import java.io.File; -import java.io.IOException; - -/** - * Firehose to give out only first n events from the delegate firehose. - */ -public class FixedCountFirehoseFactory implements FirehoseFactory -{ - private final FirehoseFactory delegate; - private final int count; - - @JsonCreator - public FixedCountFirehoseFactory( - @JsonProperty("delegate") FirehoseFactory delegate, - @JsonProperty("count") int count - ) - { - this.delegate = delegate; - this.count = count; - } - - @JsonProperty - public FirehoseFactory getDelegate() - { - return delegate; - } - - @JsonProperty - public int getCount() - { - return count; - } - - @Override - public Firehose connect(final InputRowParser parser, File temporaryDirectory) throws IOException - { - return new Firehose() - { - private int i = 0; - private final Firehose delegateFirehose = delegate.connect(parser, temporaryDirectory); - - @Override - public boolean hasMore() throws IOException - { - return i < count && delegateFirehose.hasMore(); - } - - @Nullable - @Override - public InputRow nextRow() throws IOException - { - Preconditions.checkArgument(i++ < count, "Max events limit reached."); - return delegateFirehose.nextRow(); - } - - @Override - public void close() throws IOException - { - delegateFirehose.close(); - } - }; - } -} diff --git a/server/src/main/java/org/apache/druid/segment/realtime/firehose/IngestSegmentFirehose.java b/server/src/main/java/org/apache/druid/segment/realtime/firehose/IngestSegmentFirehose.java deleted file mode 100644 index c0064a25f64c..000000000000 --- a/server/src/main/java/org/apache/druid/segment/realtime/firehose/IngestSegmentFirehose.java +++ /dev/null @@ -1,210 +0,0 @@ -/* - * Licensed to the Apache Software Foundation (ASF) under one - * or more contributor license agreements. See the NOTICE file - * distributed with this work for additional information - * regarding copyright ownership. The ASF licenses this file - * to you under the Apache License, Version 2.0 (the - * "License"); you may not use this file except in compliance - * with the License. You may obtain a copy of the License at - * - * http://www.apache.org/licenses/LICENSE-2.0 - * - * Unless required by applicable law or agreed to in writing, - * software distributed under the License is distributed on an - * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY - * KIND, either express or implied. See the License for the - * specific language governing permissions and limitations - * under the License. - */ - -package org.apache.druid.segment.realtime.firehose; - -import com.google.common.base.Function; -import com.google.common.collect.Iterables; -import com.google.common.collect.Maps; -import org.apache.druid.data.input.Firehose; -import org.apache.druid.data.input.InputRow; -import org.apache.druid.data.input.MapBasedInputRow; -import org.apache.druid.data.input.impl.TimestampSpec; -import org.apache.druid.java.util.common.DateTimes; -import org.apache.druid.java.util.common.granularity.Granularities; -import org.apache.druid.java.util.common.guava.Sequence; -import org.apache.druid.java.util.common.guava.Sequences; -import org.apache.druid.java.util.common.guava.Yielder; -import org.apache.druid.java.util.common.guava.Yielders; -import org.apache.druid.query.dimension.DefaultDimensionSpec; -import org.apache.druid.query.filter.DimFilter; -import org.apache.druid.segment.BaseLongColumnValueSelector; -import org.apache.druid.segment.BaseObjectColumnValueSelector; -import org.apache.druid.segment.Cursor; -import org.apache.druid.segment.DimensionSelector; -import org.apache.druid.segment.VirtualColumns; -import org.apache.druid.segment.column.ColumnHolder; -import org.apache.druid.segment.data.IndexedInts; -import org.apache.druid.segment.filter.Filters; -import org.apache.druid.segment.transform.TransformSpec; -import org.apache.druid.segment.transform.Transformer; - -import javax.annotation.Nullable; -import java.io.IOException; -import java.util.ArrayList; -import java.util.HashMap; -import java.util.Iterator; -import java.util.List; -import java.util.Map; - -public class IngestSegmentFirehose implements Firehose -{ - private final Transformer transformer; - private Yielder rowYielder; - - public IngestSegmentFirehose( - final List adapters, - final TransformSpec transformSpec, - final List dims, - final List metrics, - final DimFilter dimFilter - ) - { - this.transformer = transformSpec.toTransformer(); - - Sequence rows = Sequences.concat( - Iterables.transform( - adapters, - new Function>() - { - @Nullable - @Override - public Sequence apply(WindowedStorageAdapter adapter) - { - return Sequences.concat( - Sequences.map( - adapter.getAdapter().makeCursors( - Filters.toFilter(dimFilter), - adapter.getInterval(), - VirtualColumns.EMPTY, - Granularities.ALL, - false, - null - ), new Function>() - { - @Nullable - @Override - public Sequence apply(final Cursor cursor) - { - final BaseLongColumnValueSelector timestampColumnSelector = - cursor.getColumnSelectorFactory().makeColumnValueSelector(ColumnHolder.TIME_COLUMN_NAME); - - final Map dimSelectors = new HashMap<>(); - for (String dim : dims) { - final DimensionSelector dimSelector = cursor - .getColumnSelectorFactory() - .makeDimensionSelector(new DefaultDimensionSpec(dim, dim)); - // dimSelector is null if the dimension is not present - if (dimSelector != null) { - dimSelectors.put(dim, dimSelector); - } - } - - final Map metSelectors = new HashMap<>(); - for (String metric : metrics) { - final BaseObjectColumnValueSelector metricSelector = - cursor.getColumnSelectorFactory().makeColumnValueSelector(metric); - metSelectors.put(metric, metricSelector); - } - - return Sequences.simple( - new Iterable() - { - @Override - public Iterator iterator() - { - return new Iterator() - { - @Override - public boolean hasNext() - { - return !cursor.isDone(); - } - - @Override - public InputRow next() - { - final Map theEvent = Maps.newLinkedHashMap(); - final long timestamp = timestampColumnSelector.getLong(); - theEvent.put(TimestampSpec.DEFAULT_COLUMN, DateTimes.utc(timestamp)); - - for (Map.Entry dimSelector : - dimSelectors.entrySet()) { - final String dim = dimSelector.getKey(); - final DimensionSelector selector = dimSelector.getValue(); - final IndexedInts vals = selector.getRow(); - - int valsSize = vals.size(); - if (valsSize == 1) { - final String dimVal = selector.lookupName(vals.get(0)); - theEvent.put(dim, dimVal); - } else if (valsSize > 1) { - List dimVals = new ArrayList<>(valsSize); - for (int i = 0; i < valsSize; ++i) { - dimVals.add(selector.lookupName(vals.get(i))); - } - theEvent.put(dim, dimVals); - } - } - - for (Map.Entry metSelector : - metSelectors.entrySet()) { - final String metric = metSelector.getKey(); - final BaseObjectColumnValueSelector selector = metSelector.getValue(); - Object value = selector.getObject(); - if (value != null) { - theEvent.put(metric, value); - } - } - cursor.advance(); - return new MapBasedInputRow(timestamp, dims, theEvent); - } - - @Override - public void remove() - { - throw new UnsupportedOperationException("Remove Not Supported"); - } - }; - } - } - ); - } - } - ) - ); - } - } - ) - ); - rowYielder = Yielders.each(rows); - } - - @Override - public boolean hasMore() - { - return !rowYielder.isDone(); - } - - @Nullable - @Override - public InputRow nextRow() - { - final InputRow inputRow = rowYielder.get(); - rowYielder = rowYielder.next(null); - return transformer.transform(inputRow); - } - - @Override - public void close() throws IOException - { - rowYielder.close(); - } - -} diff --git a/server/src/main/java/org/apache/druid/segment/realtime/firehose/PredicateFirehose.java b/server/src/main/java/org/apache/druid/segment/realtime/firehose/PredicateFirehose.java deleted file mode 100644 index d6aadf07adc6..000000000000 --- a/server/src/main/java/org/apache/druid/segment/realtime/firehose/PredicateFirehose.java +++ /dev/null @@ -1,89 +0,0 @@ -/* - * Licensed to the Apache Software Foundation (ASF) under one - * or more contributor license agreements. See the NOTICE file - * distributed with this work for additional information - * regarding copyright ownership. The ASF licenses this file - * to you under the Apache License, Version 2.0 (the - * "License"); you may not use this file except in compliance - * with the License. You may obtain a copy of the License at - * - * http://www.apache.org/licenses/LICENSE-2.0 - * - * Unless required by applicable law or agreed to in writing, - * software distributed under the License is distributed on an - * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY - * KIND, either express or implied. See the License for the - * specific language governing permissions and limitations - * under the License. - */ - -package org.apache.druid.segment.realtime.firehose; - -import com.google.common.base.Predicate; -import org.apache.druid.data.input.Firehose; -import org.apache.druid.data.input.InputRow; -import org.apache.druid.java.util.common.logger.Logger; - -import javax.annotation.Nullable; -import java.io.IOException; - -/** - * Provides a view on a firehose that only returns rows that match a certain predicate. - * Not thread-safe. - */ -public class PredicateFirehose implements Firehose -{ - private static final Logger log = new Logger(PredicateFirehose.class); - private static final int IGNORE_THRESHOLD = 5000; - private long ignored = 0; - - private final Firehose firehose; - private final Predicate predicate; - - @Nullable - private InputRow savedInputRow = null; - - public PredicateFirehose(Firehose firehose, Predicate predicate) - { - this.firehose = firehose; - this.predicate = predicate; - } - - @Override - public boolean hasMore() throws IOException - { - if (savedInputRow != null) { - return true; - } - - while (firehose.hasMore()) { - final InputRow row = firehose.nextRow(); - if (predicate.apply(row)) { - savedInputRow = row; - return true; - } - // Do not silently discard the rows - if (ignored % IGNORE_THRESHOLD == 0) { - log.warn("[%,d] InputRow(s) ignored as they do not satisfy the predicate", ignored); - } - ignored++; - } - - return false; - } - - @Nullable - @Override - public InputRow nextRow() - { - final InputRow row = savedInputRow; - savedInputRow = null; - return row; - } - - @Override - public void close() throws IOException - { - firehose.close(); - } -} diff --git a/server/src/main/java/org/apache/druid/segment/realtime/firehose/TimedShutoffFirehoseFactory.java b/server/src/main/java/org/apache/druid/segment/realtime/firehose/TimedShutoffFirehoseFactory.java deleted file mode 100644 index 9bfda42c8aac..000000000000 --- a/server/src/main/java/org/apache/druid/segment/realtime/firehose/TimedShutoffFirehoseFactory.java +++ /dev/null @@ -1,139 +0,0 @@ -/* - * Licensed to the Apache Software Foundation (ASF) under one - * or more contributor license agreements. See the NOTICE file - * distributed with this work for additional information - * regarding copyright ownership. The ASF licenses this file - * to you under the Apache License, Version 2.0 (the - * "License"); you may not use this file except in compliance - * with the License. You may obtain a copy of the License at - * - * http://www.apache.org/licenses/LICENSE-2.0 - * - * Unless required by applicable law or agreed to in writing, - * software distributed under the License is distributed on an - * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY - * KIND, either express or implied. See the License for the - * specific language governing permissions and limitations - * under the License. - */ - -package org.apache.druid.segment.realtime.firehose; - -import com.fasterxml.jackson.annotation.JsonCreator; -import com.fasterxml.jackson.annotation.JsonProperty; -import com.google.errorprone.annotations.concurrent.GuardedBy; -import org.apache.druid.data.input.Firehose; -import org.apache.druid.data.input.FirehoseFactory; -import org.apache.druid.data.input.InputRow; -import org.apache.druid.data.input.impl.InputRowParser; -import org.apache.druid.java.util.common.concurrent.Execs; -import org.apache.druid.java.util.emitter.EmittingLogger; -import org.apache.druid.utils.CloseableUtils; -import org.joda.time.DateTime; - -import javax.annotation.Nullable; -import java.io.File; -import java.io.IOException; -import java.util.concurrent.ScheduledExecutorService; -import java.util.concurrent.TimeUnit; - -/** - * Creates firehoses that shut off at a particular time. Useful for limiting the lifespan of a realtime job. - * - * Each {@link Firehose} created by this factory spins up and manages one thread for calling {@link Firehose#close()} - * asynchronously at the specified {@link #shutoffTime}. - */ -@Deprecated -public class TimedShutoffFirehoseFactory implements FirehoseFactory -{ - private static final EmittingLogger log = new EmittingLogger(FirehoseFactory.class); - - private final FirehoseFactory delegateFactory; - private final DateTime shutoffTime; - - @JsonCreator - public TimedShutoffFirehoseFactory( - @JsonProperty("delegate") FirehoseFactory delegateFactory, - @JsonProperty("shutoffTime") DateTime shutoffTime - ) - { - this.delegateFactory = delegateFactory; - this.shutoffTime = shutoffTime; - } - - @Override - public Firehose connect(InputRowParser parser, File temporaryDirectory) throws IOException - { - return new TimedShutoffFirehose(parser, temporaryDirectory); - } - - class TimedShutoffFirehose implements Firehose - { - private final Firehose firehose; - private final ScheduledExecutorService shutdownExec; - @GuardedBy("this") - private boolean closed = false; - - TimedShutoffFirehose(InputRowParser parser, File temporaryDirectory) throws IOException - { - firehose = delegateFactory.connect(parser, temporaryDirectory); - - shutdownExec = Execs.scheduledSingleThreaded("timed-shutoff-firehose-%d"); - - shutdownExec.schedule( - () -> { - log.info("Closing delegate firehose."); - - try { - TimedShutoffFirehose.this.close(); - } - catch (IOException e) { - log.warn(e, "Failed to close delegate firehose, ignoring."); - } - }, - shutoffTime.getMillis() - System.currentTimeMillis(), - TimeUnit.MILLISECONDS - ); - - log.info("Firehose created, will shut down at: %s", shutoffTime); - } - - @Override - public boolean hasMore() throws IOException - { - return firehose.hasMore(); - } - - @Nullable - @Override - public InputRow nextRow() throws IOException - { - return firehose.nextRow(); - } - - /** - * This method is synchronized because it might be called concurrently from multiple threads: from {@link - * #shutdownExec}, and explicitly on this Firehose object. - */ - @Override - public synchronized void close() throws IOException - { - if (!closed) { - closed = true; - CloseableUtils.closeAll(firehose, shutdownExec::shutdownNow); - } - } - } - - @JsonProperty("delegate") - public FirehoseFactory getDelegateFactory() - { - return delegateFactory; - } - - @JsonProperty("shutoffTime") - public DateTime getShutoffTime() - { - return shutoffTime; - } -} diff --git a/server/src/main/java/org/apache/druid/segment/realtime/firehose/package-info.java b/server/src/main/java/org/apache/druid/segment/realtime/package-info.java similarity index 94% rename from server/src/main/java/org/apache/druid/segment/realtime/firehose/package-info.java rename to server/src/main/java/org/apache/druid/segment/realtime/package-info.java index e4a06999f363..2f80f8a1a06c 100644 --- a/server/src/main/java/org/apache/druid/segment/realtime/firehose/package-info.java +++ b/server/src/main/java/org/apache/druid/segment/realtime/package-info.java @@ -18,6 +18,6 @@ */ @EverythingIsNonnullByDefault -package org.apache.druid.segment.realtime.firehose; +package org.apache.druid.segment.realtime; import org.apache.druid.annotations.EverythingIsNonnullByDefault; diff --git a/server/src/main/java/org/apache/druid/server/initialization/jetty/ChatHandlerServerModule.java b/server/src/main/java/org/apache/druid/server/initialization/jetty/ChatHandlerServerModule.java index aead7b86c719..8ee3d5ba0814 100644 --- a/server/src/main/java/org/apache/druid/server/initialization/jetty/ChatHandlerServerModule.java +++ b/server/src/main/java/org/apache/druid/server/initialization/jetty/ChatHandlerServerModule.java @@ -31,7 +31,7 @@ import org.apache.druid.guice.annotations.RemoteChatHandler; import org.apache.druid.guice.annotations.Self; import org.apache.druid.java.util.common.lifecycle.Lifecycle; -import org.apache.druid.segment.realtime.firehose.ChatHandlerResource; +import org.apache.druid.segment.realtime.ChatHandlerResource; import org.apache.druid.server.DruidNode; import org.apache.druid.server.initialization.ServerConfig; import org.apache.druid.server.initialization.TLSServerConfig; diff --git a/server/src/main/java/org/apache/druid/server/initialization/jetty/CliIndexerServerModule.java b/server/src/main/java/org/apache/druid/server/initialization/jetty/CliIndexerServerModule.java index 687ca2ef5485..feb61965daa3 100644 --- a/server/src/main/java/org/apache/druid/server/initialization/jetty/CliIndexerServerModule.java +++ b/server/src/main/java/org/apache/druid/server/initialization/jetty/CliIndexerServerModule.java @@ -32,7 +32,7 @@ import org.apache.druid.guice.annotations.Self; import org.apache.druid.java.util.common.lifecycle.Lifecycle; import org.apache.druid.query.lookup.LookupModule; -import org.apache.druid.segment.realtime.firehose.ChatHandlerResource; +import org.apache.druid.segment.realtime.ChatHandlerResource; import org.apache.druid.server.DruidNode; import org.apache.druid.server.initialization.ServerConfig; import org.apache.druid.server.initialization.TLSServerConfig; diff --git a/server/src/main/java/org/apache/druid/server/initialization/jetty/TaskIdResponseHeaderFilterHolder.java b/server/src/main/java/org/apache/druid/server/initialization/jetty/TaskIdResponseHeaderFilterHolder.java index 7dcd3b2c237a..c4c0cb8ce4b0 100644 --- a/server/src/main/java/org/apache/druid/server/initialization/jetty/TaskIdResponseHeaderFilterHolder.java +++ b/server/src/main/java/org/apache/druid/server/initialization/jetty/TaskIdResponseHeaderFilterHolder.java @@ -21,7 +21,7 @@ import com.google.common.collect.ImmutableMap; import org.apache.druid.java.util.common.StringUtils; -import org.apache.druid.segment.realtime.firehose.ChatHandlerResource; +import org.apache.druid.segment.realtime.ChatHandlerResource; public class TaskIdResponseHeaderFilterHolder extends ResponseHeaderFilterHolder { diff --git a/server/src/main/java/org/apache/druid/server/metrics/EventReceiverFirehoseMetric.java b/server/src/main/java/org/apache/druid/server/metrics/EventReceiverFirehoseMetric.java deleted file mode 100644 index 13502e18493c..000000000000 --- a/server/src/main/java/org/apache/druid/server/metrics/EventReceiverFirehoseMetric.java +++ /dev/null @@ -1,48 +0,0 @@ -/* - * Licensed to the Apache Software Foundation (ASF) under one - * or more contributor license agreements. See the NOTICE file - * distributed with this work for additional information - * regarding copyright ownership. The ASF licenses this file - * to you under the Apache License, Version 2.0 (the - * "License"); you may not use this file except in compliance - * with the License. You may obtain a copy of the License at - * - * http://www.apache.org/licenses/LICENSE-2.0 - * - * Unless required by applicable law or agreed to in writing, - * software distributed under the License is distributed on an - * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY - * KIND, either express or implied. See the License for the - * specific language governing permissions and limitations - * under the License. - */ - -package org.apache.druid.server.metrics; - -/** - * An EventReceiverFirehoseMetric is an object with metrics about EventReceiverFirehose objects. - * It is not likely that anything other than an EventReceiverFirehose actually implements this. - * This interface is not part of the public API and backwards incompatible changes can occur without - * requiring a major (or even minor) version change. - * The interface's primary purpose is to be able to share metrics via the EventReceiverFirehoseRegister - * without exposing the entire EventReceiverFirehose - */ -public interface EventReceiverFirehoseMetric -{ - /** - * Return the current number of {@link org.apache.druid.data.input.InputRow} that are stored in the buffer. - */ - int getCurrentBufferSize(); - - /** - * Return the capacity of the buffer. - */ - int getCapacity(); - - /** - * Return the number of bytes received by the firehose. - */ - long getBytesReceived(); - - -} diff --git a/server/src/main/java/org/apache/druid/server/metrics/EventReceiverFirehoseMonitor.java b/server/src/main/java/org/apache/druid/server/metrics/EventReceiverFirehoseMonitor.java deleted file mode 100644 index 29c1808ad515..000000000000 --- a/server/src/main/java/org/apache/druid/server/metrics/EventReceiverFirehoseMonitor.java +++ /dev/null @@ -1,90 +0,0 @@ -/* - * Licensed to the Apache Software Foundation (ASF) under one - * or more contributor license agreements. See the NOTICE file - * distributed with this work for additional information - * regarding copyright ownership. The ASF licenses this file - * to you under the Apache License, Version 2.0 (the - * "License"); you may not use this file except in compliance - * with the License. You may obtain a copy of the License at - * - * http://www.apache.org/licenses/LICENSE-2.0 - * - * Unless required by applicable law or agreed to in writing, - * software distributed under the License is distributed on an - * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY - * KIND, either express or implied. See the License for the - * specific language governing permissions and limitations - * under the License. - */ - -package org.apache.druid.server.metrics; - -import com.google.common.collect.ImmutableMap; -import com.google.common.collect.Lists; -import com.google.inject.Inject; -import org.apache.druid.java.util.emitter.service.ServiceEmitter; -import org.apache.druid.java.util.emitter.service.ServiceMetricEvent; -import org.apache.druid.java.util.metrics.AbstractMonitor; -import org.apache.druid.java.util.metrics.KeyedDiff; -import org.apache.druid.java.util.metrics.MonitorUtils; -import org.apache.druid.query.DruidMetrics; - -import java.util.Map; -import java.util.Properties; - -public class EventReceiverFirehoseMonitor extends AbstractMonitor -{ - - private final EventReceiverFirehoseRegister register; - private final KeyedDiff keyedDiff = new KeyedDiff(); - private final Map dimensions; - - @Inject - public EventReceiverFirehoseMonitor( - EventReceiverFirehoseRegister eventReceiverFirehoseRegister, - Properties props - ) - { - this.register = eventReceiverFirehoseRegister; - this.dimensions = MonitorsConfig.extractDimensions( - props, - Lists.newArrayList(DruidMetrics.DATASOURCE, DruidMetrics.TASK_ID, DruidMetrics.TASK_TYPE) - ); - } - - @Override - public boolean doMonitor(ServiceEmitter emitter) - { - for (Map.Entry entry : register.getMetrics()) { - final String serviceName = entry.getKey(); - final EventReceiverFirehoseMetric metric = entry.getValue(); - - final ServiceMetricEvent.Builder builder = createEventBuilder(serviceName) - .setDimension( - "bufferCapacity", - String.valueOf(metric.getCapacity()) - ); - emitter.emit(builder.setMetric("ingest/events/buffered", metric.getCurrentBufferSize())); - Map diff = keyedDiff.to( - serviceName, - ImmutableMap.of("ingest/bytes/received", metric.getBytesReceived()) - ); - if (diff != null) { - final ServiceMetricEvent.Builder eventBuilder = createEventBuilder(serviceName); - for (Map.Entry diffEntry : diff.entrySet()) { - emitter.emit(eventBuilder.setMetric(diffEntry.getKey(), diffEntry.getValue())); - } - } - } - - return true; - } - - private ServiceMetricEvent.Builder createEventBuilder(String serviceName) - { - ServiceMetricEvent.Builder builder = ServiceMetricEvent.builder() - .setDimension("serviceName", serviceName); - MonitorUtils.addDimensionsToBuilder(builder, dimensions); - return builder; - } -} diff --git a/server/src/main/java/org/apache/druid/server/metrics/EventReceiverFirehoseRegister.java b/server/src/main/java/org/apache/druid/server/metrics/EventReceiverFirehoseRegister.java deleted file mode 100644 index 66a022992e74..000000000000 --- a/server/src/main/java/org/apache/druid/server/metrics/EventReceiverFirehoseRegister.java +++ /dev/null @@ -1,56 +0,0 @@ -/* - * Licensed to the Apache Software Foundation (ASF) under one - * or more contributor license agreements. See the NOTICE file - * distributed with this work for additional information - * regarding copyright ownership. The ASF licenses this file - * to you under the Apache License, Version 2.0 (the - * "License"); you may not use this file except in compliance - * with the License. You may obtain a copy of the License at - * - * http://www.apache.org/licenses/LICENSE-2.0 - * - * Unless required by applicable law or agreed to in writing, - * software distributed under the License is distributed on an - * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY - * KIND, either express or implied. See the License for the - * specific language governing permissions and limitations - * under the License. - */ - -package org.apache.druid.server.metrics; - -import org.apache.druid.java.util.common.ISE; -import org.apache.druid.java.util.common.logger.Logger; - -import java.util.Map; -import java.util.concurrent.ConcurrentHashMap; -import java.util.concurrent.ConcurrentMap; - -public class EventReceiverFirehoseRegister -{ - - private static final Logger log = new Logger(EventReceiverFirehoseRegister.class); - - private final ConcurrentMap metrics = new ConcurrentHashMap<>(); - - public void register(String serviceName, EventReceiverFirehoseMetric metric) - { - log.info("Registering EventReceiverFirehoseMetric for service [%s]", serviceName); - if (metrics.putIfAbsent(serviceName, metric) != null) { - throw new ISE("Service [%s] is already registered!", serviceName); - } - } - - public Iterable> getMetrics() - { - return metrics.entrySet(); - } - - public void unregister(String serviceName) - { - log.info("Unregistering EventReceiverFirehoseMetric for service [%s]", serviceName); - if (metrics.remove(serviceName) == null) { - log.warn("Unregistering a non-exist service. Service [%s] never exists.", serviceName); - } - } -} diff --git a/server/src/main/java/org/apache/druid/server/metrics/MetricsModule.java b/server/src/main/java/org/apache/druid/server/metrics/MetricsModule.java index 278a170910ad..ab2cdb3811f0 100644 --- a/server/src/main/java/org/apache/druid/server/metrics/MetricsModule.java +++ b/server/src/main/java/org/apache/druid/server/metrics/MetricsModule.java @@ -91,7 +91,6 @@ public void configure(Binder binder) binder.bind(DataSourceTaskIdHolder.class).in(LazySingleton.class); - binder.bind(EventReceiverFirehoseRegister.class).in(LazySingleton.class); binder.bind(ExecutorServiceMonitor.class).in(LazySingleton.class); // Instantiate eagerly so that we get everything registered and put into the Lifecycle diff --git a/server/src/test/java/org/apache/druid/curator/discovery/ServiceAnnouncerTest.java b/server/src/test/java/org/apache/druid/curator/discovery/ServiceAnnouncerTest.java index 57f7517239d6..f80f35bdca83 100644 --- a/server/src/test/java/org/apache/druid/curator/discovery/ServiceAnnouncerTest.java +++ b/server/src/test/java/org/apache/druid/curator/discovery/ServiceAnnouncerTest.java @@ -19,7 +19,6 @@ package org.apache.druid.curator.discovery; -import com.google.common.base.Predicate; import com.google.common.collect.ImmutableList; import com.google.common.collect.Iterators; import org.apache.curator.x.discovery.ServiceDiscovery; @@ -49,27 +48,21 @@ public void testServiceAnnouncement() throws Exception curator.blockUntilConnected(); List serviceNames = ImmutableList.of( "druid/overlord", - "druid/coordinator", - "druid/firehose/tranquility_test-50-0000-0000" + "druid/coordinator" ); final ServiceDiscovery serviceDiscovery = createAndAnnounceServices(serviceNames); Assert.assertTrue( Iterators.all( serviceNames.iterator(), - new Predicate() - { - @Override - public boolean apply(String input) - { - try { - return serviceDiscovery.queryForInstances(input.replace('/', ':')).size() == 1; - } - catch (Exception e) { - throw new ISE( - "Something went wrong while finding instance with name [%s] in Service Discovery", - input - ); - } + input -> { + try { + return serviceDiscovery.queryForInstances(input.replace('/', ':')).size() == 1; + } + catch (Exception e) { + throw new ISE( + "Something went wrong while finding instance with name [%s] in Service Discovery", + input + ); } } ) diff --git a/server/src/test/java/org/apache/druid/guice/FirehoseModuleTest.java b/server/src/test/java/org/apache/druid/guice/FirehoseModuleTest.java deleted file mode 100644 index 8ecc93dece2b..000000000000 --- a/server/src/test/java/org/apache/druid/guice/FirehoseModuleTest.java +++ /dev/null @@ -1,93 +0,0 @@ -/* - * Licensed to the Apache Software Foundation (ASF) under one - * or more contributor license agreements. See the NOTICE file - * distributed with this work for additional information - * regarding copyright ownership. The ASF licenses this file - * to you under the Apache License, Version 2.0 (the - * "License"); you may not use this file except in compliance - * with the License. You may obtain a copy of the License at - * - * http://www.apache.org/licenses/LICENSE-2.0 - * - * Unless required by applicable law or agreed to in writing, - * software distributed under the License is distributed on an - * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY - * KIND, either express or implied. See the License for the - * specific language governing permissions and limitations - * under the License. - */ - -package org.apache.druid.guice; - -import com.fasterxml.jackson.databind.Module; -import com.fasterxml.jackson.databind.ObjectMapper; -import com.fasterxml.jackson.databind.cfg.MapperConfig; -import com.fasterxml.jackson.databind.introspect.AnnotatedClass; -import com.fasterxml.jackson.databind.jsontype.NamedType; -import com.google.common.reflect.ClassPath; -import org.apache.druid.data.input.FirehoseFactory; -import org.apache.druid.segment.realtime.firehose.ClippedFirehoseFactory; -import org.apache.druid.utils.JvmUtils; -import org.junit.Assert; -import org.junit.Test; - -import java.io.IOException; -import java.lang.reflect.Modifier; -import java.net.URL; -import java.net.URLClassLoader; -import java.util.Collection; -import java.util.Set; -import java.util.function.Predicate; -import java.util.stream.Collectors; - -public class FirehoseModuleTest -{ - private static final Predicate IS_FIREHOSE_FACTORY = - c -> FirehoseFactory.class.isAssignableFrom(c) && !Modifier.isAbstract(c.getModifiers()); - - @Test - public void testAllFirehoseFactorySubtypesRegistered() throws IOException - { - ObjectMapper objectMapper = createObjectMapper(); - Set registeredSubtypeClasses = getFirehoseFactorySubtypeClasses(objectMapper); - String packageName = ClippedFirehoseFactory.class.getPackage().getName(); - Set expectedSubtypeClasses = getFirehoseFactoryClassesInPackage(packageName); - Assert.assertEquals(expectedSubtypeClasses, registeredSubtypeClasses); - } - - private static ObjectMapper createObjectMapper() - { - ObjectMapper objectMapper = new ObjectMapper(); - for (Module jacksonModule : new FirehoseModule().getJacksonModules()) { - objectMapper.registerModule(jacksonModule); - } - return objectMapper; - } - - private static Set getFirehoseFactorySubtypeClasses(ObjectMapper objectMapper) - { - Class parentClass = FirehoseFactory.class; - MapperConfig config = objectMapper.getDeserializationConfig(); - AnnotatedClass ac = AnnotatedClass.constructWithoutSuperTypes(parentClass, config); - Collection subtypes = objectMapper.getSubtypeResolver().collectAndResolveSubtypesByClass(config, ac); - Assert.assertNotNull(subtypes); - return subtypes.stream() - .map(NamedType::getType) - .filter(c -> !c.equals(parentClass)) - .collect(Collectors.toSet()); - } - - @SuppressWarnings("UnstableApiUsage") // for ClassPath - private static Set getFirehoseFactoryClassesInPackage(String packageName) throws IOException - { - // workaround for Guava 16, which can only parse the classpath from URLClassLoaders - // requires Guava 28 or later to work properly with the system class loader in Java 9 and above - URLClassLoader classloader = new URLClassLoader(JvmUtils.systemClassPath().toArray(new URL[0])); - ClassPath classPath = ClassPath.from(classloader); - return classPath.getTopLevelClasses(packageName).stream() - .map(ClassPath.ClassInfo::load) - .filter(IS_FIREHOSE_FACTORY) - .collect(Collectors.toSet()); - } -} - diff --git a/server/src/test/java/org/apache/druid/metadata/input/SqlEntityTest.java b/server/src/test/java/org/apache/druid/metadata/input/SqlEntityTest.java index ccd71cfaff5e..4053ec9ecf22 100644 --- a/server/src/test/java/org/apache/druid/metadata/input/SqlEntityTest.java +++ b/server/src/test/java/org/apache/druid/metadata/input/SqlEntityTest.java @@ -68,7 +68,7 @@ public void testExecuteQuery() throws IOException File tmpFile = File.createTempFile("testQueryResults", ""); InputEntity.CleanableFile queryResult = SqlEntity.openCleanableFile( VALID_SQL, - testUtils.getDerbyFirehoseConnector(), + testUtils.getDerbyInputSourceConnector(), mapper, true, tmpFile @@ -95,7 +95,7 @@ public void testFileDeleteOnInvalidQuery() throws IOException IOException.class, () -> SqlEntity.openCleanableFile( INVALID_SQL, - testUtils.getDerbyFirehoseConnector(), + testUtils.getDerbyInputSourceConnector(), mapper, true, tmpFile diff --git a/server/src/test/java/org/apache/druid/metadata/input/SqlInputSourceTest.java b/server/src/test/java/org/apache/druid/metadata/input/SqlInputSourceTest.java index 7a5ea7b21490..c48dfd839463 100644 --- a/server/src/test/java/org/apache/druid/metadata/input/SqlInputSourceTest.java +++ b/server/src/test/java/org/apache/druid/metadata/input/SqlInputSourceTest.java @@ -41,7 +41,7 @@ import org.apache.druid.java.util.common.FileUtils; import org.apache.druid.java.util.common.parsers.CloseableIterator; import org.apache.druid.metadata.MetadataStorageConnectorConfig; -import org.apache.druid.metadata.SQLFirehoseDatabaseConnector; +import org.apache.druid.metadata.SQLInputSourceDatabaseConnector; import org.apache.druid.metadata.TestDerbyConnector; import org.apache.druid.segment.TestHelper; import org.apache.druid.server.initialization.JdbcAccessSecurityConfig; @@ -66,7 +66,7 @@ public class SqlInputSourceTest { - private static final List FIREHOSE_TMP_DIRS = new ArrayList<>(); + private static final List INPUT_SOURCE_TMP_DIRS = new ArrayList<>(); private final String TABLE_1 = "FOOS_TABLE_1"; private final String TABLE_2 = "FOOS_TABLE_2"; @@ -94,31 +94,31 @@ public void setUp() @AfterClass public static void teardown() throws IOException { - for (File dir : FIREHOSE_TMP_DIRS) { + for (File dir : INPUT_SOURCE_TMP_DIRS) { org.apache.commons.io.FileUtils.forceDelete(dir); } } - private File createFirehoseTmpDir(String dirSuffix) throws IOException + private File createInputSourceTmpDir(String dirSuffix) throws IOException { - final File firehoseTempDir = File.createTempFile( + final File inputSourceTempDir = File.createTempFile( SqlInputSourceTest.class.getSimpleName(), dirSuffix ); - org.apache.commons.io.FileUtils.forceDelete(firehoseTempDir); - FileUtils.mkdirp(firehoseTempDir); - FIREHOSE_TMP_DIRS.add(firehoseTempDir); - return firehoseTempDir; + org.apache.commons.io.FileUtils.forceDelete(inputSourceTempDir); + FileUtils.mkdirp(inputSourceTempDir); + INPUT_SOURCE_TMP_DIRS.add(inputSourceTempDir); + return inputSourceTempDir; } @Test public void testSerde() throws IOException { - mapper.registerSubtypes(TestSerdeFirehoseConnector.class); - final SqlInputSourceTest.TestSerdeFirehoseConnector testSerdeFirehoseConnector = new SqlInputSourceTest.TestSerdeFirehoseConnector( + mapper.registerSubtypes(TestSerdeInputSourceConnector.class); + final TestSerdeInputSourceConnector serdeInputSourceConnector = new TestSerdeInputSourceConnector( new MetadataStorageConnectorConfig()); final SqlInputSource sqlInputSource = - new SqlInputSource(SqlTestUtils.selectFrom(TABLE_1), true, testSerdeFirehoseConnector, mapper); + new SqlInputSource(SqlTestUtils.selectFrom(TABLE_1), true, serdeInputSourceConnector, mapper); final String valueString = mapper.writeValueAsString(sqlInputSource); final SqlInputSource inputSourceFromJson = mapper.readValue(valueString, SqlInputSource.class); Assert.assertEquals(sqlInputSource, inputSourceFromJson); @@ -127,11 +127,11 @@ public void testSerde() throws IOException @Test public void testGetTypes() { - mapper.registerSubtypes(TestSerdeFirehoseConnector.class); - final SqlInputSourceTest.TestSerdeFirehoseConnector testSerdeFirehoseConnector = new SqlInputSourceTest.TestSerdeFirehoseConnector( + mapper.registerSubtypes(TestSerdeInputSourceConnector.class); + final TestSerdeInputSourceConnector serdeInputSourceConnector = new TestSerdeInputSourceConnector( new MetadataStorageConnectorConfig()); final SqlInputSource sqlInputSource = - new SqlInputSource(SqlTestUtils.selectFrom(TABLE_1), true, testSerdeFirehoseConnector, mapper); + new SqlInputSource(SqlTestUtils.selectFrom(TABLE_1), true, serdeInputSourceConnector, mapper); Assert.assertEquals(Collections.singleton(SqlInputSource.TYPE_KEY), sqlInputSource.getTypes()); } @@ -141,13 +141,13 @@ public void testSingleSplit() throws Exception derbyConnector = derbyConnectorRule.getConnector(); SqlTestUtils testUtils = new SqlTestUtils(derbyConnector); final List expectedRows = testUtils.createTableWithRows(TABLE_1, 10); - final File tempDir = createFirehoseTmpDir("testSingleSplit"); + final File tempDir = createInputSourceTmpDir("testSingleSplit"); final InputStats inputStats = new InputStatsImpl(); SqlInputSource sqlInputSource = new SqlInputSource( SqlTestUtils.selectFrom(TABLE_1), true, - testUtils.getDerbyFirehoseConnector(), + testUtils.getDerbyInputSourceConnector(), mapper ); InputSourceReader sqlReader = sqlInputSource.fixedFormatReader(INPUT_ROW_SCHEMA, tempDir); @@ -170,11 +170,11 @@ public void testMultipleSplits() throws Exception SqlTestUtils testUtils = new SqlTestUtils(derbyConnector); final List expectedRowsTable1 = testUtils.createTableWithRows(TABLE_1, 10); final List expectedRowsTable2 = testUtils.createTableWithRows(TABLE_2, 10); - final File tempDir = createFirehoseTmpDir("testMultipleSplit"); + final File tempDir = createInputSourceTmpDir("testMultipleSplit"); SqlInputSource sqlInputSource = new SqlInputSource( SqlTestUtils.selectFrom(TABLE_1, TABLE_2), true, - testUtils.getDerbyFirehoseConnector(), + testUtils.getDerbyInputSourceConnector(), mapper ); @@ -198,7 +198,7 @@ public void testNumSplits() SqlTestUtils testUtils = new SqlTestUtils(derbyConnector); final List sqls = SqlTestUtils.selectFrom(TABLE_1, TABLE_2); SqlInputSource sqlInputSource = - new SqlInputSource(sqls, true, testUtils.getDerbyFirehoseConnector(), mapper); + new SqlInputSource(sqls, true, testUtils.getDerbyInputSourceConnector(), mapper); InputFormat inputFormat = EasyMock.createMock(InputFormat.class); Stream> sqlSplits = sqlInputSource.createSplits(inputFormat, null); Assert.assertEquals(sqls, sqlSplits.map(InputSplit::get).collect(Collectors.toList())); @@ -212,9 +212,9 @@ public void testSample() throws Exception SqlTestUtils testUtils = new SqlTestUtils(derbyConnector); final List expectedRows = testUtils.createTableWithRows(TABLE_1, 10); try { - final File tempDir = createFirehoseTmpDir("testSingleSplit"); + final File tempDir = createInputSourceTmpDir("testSingleSplit"); SqlInputSource sqlInputSource = - new SqlInputSource(SqlTestUtils.selectFrom(TABLE_1), true, testUtils.getDerbyFirehoseConnector(), mapper); + new SqlInputSource(SqlTestUtils.selectFrom(TABLE_1), true, testUtils.getDerbyInputSourceConnector(), mapper); InputSourceReader sqlReader = sqlInputSource.fixedFormatReader(INPUT_ROW_SCHEMA, tempDir); CloseableIterator resultIterator = sqlReader.sample(); final List rows = new ArrayList<>(); @@ -230,6 +230,64 @@ public void testSample() throws Exception } } + @Test + public void testConnectorValidationInvalidUri() + { + derbyConnector = derbyConnectorRule.getConnector(); + Throwable t = Assert.assertThrows( + IllegalArgumentException.class, + () -> new SqlTestUtils( + derbyConnector, + new MetadataStorageConnectorConfig() + { + @Override + public String getConnectURI() + { + return ""; + } + } + ) + ); + Assert.assertEquals("connectURI cannot be null or empty", t.getMessage()); + } + + @Test + public void testConnectorValidationAllowedProperties() + { + derbyConnector = derbyConnectorRule.getConnector(); + Throwable t = Assert.assertThrows( + IllegalArgumentException.class, + () -> new SqlTestUtils( + derbyConnector, + new MetadataStorageConnectorConfig(), + new JdbcAccessSecurityConfig() + ) + ); + Assert.assertEquals( + "The property [user] is not in the allowed list [useSSL, requireSSL, ssl, sslmode]", + t.getMessage() + ); + } + + @Test + public void testConnectorValidationSkipAllowedProperties() + { + derbyConnector = derbyConnectorRule.getConnector(); + SqlTestUtils testUtils = new SqlTestUtils( + derbyConnector, + new MetadataStorageConnectorConfig(), + new JdbcAccessSecurityConfig() + { + @Override + public boolean isEnforceAllowedProperties() + { + return false; + } + } + ); + Assert.assertNotNull(testUtils); + } + @Test public void testEquals() { @@ -240,18 +298,19 @@ public void testEquals() new ObjectMapper() ) .withIgnoredFields("objectMapper") - .withNonnullFields("sqls", "sqlFirehoseDatabaseConnector") + .withNonnullFields("sqls", "sqlInputSourceDatabaseConnector") .usingGetClass() .verify(); } + @JsonTypeName("test") - private static class TestSerdeFirehoseConnector extends SQLFirehoseDatabaseConnector + private static class TestSerdeInputSourceConnector extends SQLInputSourceDatabaseConnector { private final DBI dbi; private final MetadataStorageConnectorConfig metadataStorageConnectorConfig; - private TestSerdeFirehoseConnector( + private TestSerdeInputSourceConnector( @JsonProperty("connectorConfig") MetadataStorageConnectorConfig metadataStorageConnectorConfig ) { @@ -287,7 +346,7 @@ public boolean equals(Object o) if (o == null || getClass() != o.getClass()) { return false; } - TestSerdeFirehoseConnector that = (TestSerdeFirehoseConnector) o; + TestSerdeInputSourceConnector that = (TestSerdeInputSourceConnector) o; return metadataStorageConnectorConfig.equals(that.metadataStorageConnectorConfig); } diff --git a/server/src/test/java/org/apache/druid/metadata/input/SqlTestUtils.java b/server/src/test/java/org/apache/druid/metadata/input/SqlTestUtils.java index 2e99bfbb301b..f999e6dd41c4 100644 --- a/server/src/test/java/org/apache/druid/metadata/input/SqlTestUtils.java +++ b/server/src/test/java/org/apache/druid/metadata/input/SqlTestUtils.java @@ -28,7 +28,7 @@ import org.apache.druid.java.util.common.DateTimes; import org.apache.druid.java.util.common.StringUtils; import org.apache.druid.metadata.MetadataStorageConnectorConfig; -import org.apache.druid.metadata.SQLFirehoseDatabaseConnector; +import org.apache.druid.metadata.SQLInputSourceDatabaseConnector; import org.apache.druid.metadata.TestDerbyConnector; import org.apache.druid.server.initialization.JdbcAccessSecurityConfig; import org.junit.Rule; @@ -49,24 +49,48 @@ public class SqlTestUtils { @Rule public final TestDerbyConnector.DerbyConnectorRule derbyConnectorRule = new TestDerbyConnector.DerbyConnectorRule(); - private final TestDerbyFirehoseConnector derbyFirehoseConnector; + private final TestDerbyInputSourceConnector derbyInputSourceConnector; private final TestDerbyConnector derbyConnector; public SqlTestUtils(TestDerbyConnector derbyConnector) { this.derbyConnector = derbyConnector; - this.derbyFirehoseConnector = new SqlTestUtils.TestDerbyFirehoseConnector( + this.derbyInputSourceConnector = new TestDerbyInputSourceConnector( new MetadataStorageConnectorConfig(), derbyConnector.getDBI() ); } - private static class TestDerbyFirehoseConnector extends SQLFirehoseDatabaseConnector + public SqlTestUtils(TestDerbyConnector derbyConnector, MetadataStorageConnectorConfig config) + { + this.derbyConnector = derbyConnector; + this.derbyInputSourceConnector = new TestDerbyInputSourceConnector( + config, + derbyConnector.getDBI() + ); + } + + public SqlTestUtils( + TestDerbyConnector derbyConnector, + MetadataStorageConnectorConfig config, + JdbcAccessSecurityConfig securityConfig + ) + { + this.derbyConnector = derbyConnector; + this.derbyInputSourceConnector = new TestDerbyInputSourceConnector( + config, + securityConfig, + derbyConnector.getDBI() + ); + } + + private static class TestDerbyInputSourceConnector extends SQLInputSourceDatabaseConnector { private final DBI dbi; - private TestDerbyFirehoseConnector( - @JsonProperty("connectorConfig") MetadataStorageConnectorConfig metadataStorageConnectorConfig, DBI dbi + private TestDerbyInputSourceConnector( + @JsonProperty("connectorConfig") MetadataStorageConnectorConfig metadataStorageConnectorConfig, + DBI dbi ) { final BasicDataSource datasource = getDatasource( @@ -85,6 +109,21 @@ public Set getAllowedProperties() this.dbi = dbi; } + private TestDerbyInputSourceConnector( + MetadataStorageConnectorConfig metadataStorageConnectorConfig, + JdbcAccessSecurityConfig securityConfig, + DBI dbi + ) + { + final BasicDataSource datasource = getDatasource( + metadataStorageConnectorConfig, + securityConfig + ); + datasource.setDriverClassLoader(getClass().getClassLoader()); + datasource.setDriverClassName("org.apache.derby.jdbc.ClientDriver"); + this.dbi = dbi; + } + @Override public DBI getDBI() { @@ -151,9 +190,9 @@ public void dropTable(final String tableName) ); } - public TestDerbyFirehoseConnector getDerbyFirehoseConnector() + public TestDerbyInputSourceConnector getDerbyInputSourceConnector() { - return derbyFirehoseConnector; + return derbyInputSourceConnector; } /** diff --git a/server/src/test/java/org/apache/druid/segment/realtime/firehose/ChatHandlerResourceTest.java b/server/src/test/java/org/apache/druid/segment/realtime/ChatHandlerResourceTest.java similarity index 97% rename from server/src/test/java/org/apache/druid/segment/realtime/firehose/ChatHandlerResourceTest.java rename to server/src/test/java/org/apache/druid/segment/realtime/ChatHandlerResourceTest.java index 870636fb416a..21d99c622255 100644 --- a/server/src/test/java/org/apache/druid/segment/realtime/firehose/ChatHandlerResourceTest.java +++ b/server/src/test/java/org/apache/druid/segment/realtime/ChatHandlerResourceTest.java @@ -18,7 +18,7 @@ */ -package org.apache.druid.segment.realtime.firehose; +package org.apache.druid.segment.realtime; import com.google.common.base.Optional; import org.apache.druid.server.initialization.jetty.ServiceUnavailableException; diff --git a/server/src/test/java/org/apache/druid/segment/realtime/firehose/ServiceAnnouncingChatHandlerProviderTest.java b/server/src/test/java/org/apache/druid/segment/realtime/ServiceAnnouncingChatHandlerProviderTest.java similarity index 99% rename from server/src/test/java/org/apache/druid/segment/realtime/firehose/ServiceAnnouncingChatHandlerProviderTest.java rename to server/src/test/java/org/apache/druid/segment/realtime/ServiceAnnouncingChatHandlerProviderTest.java index 71a3fe308f4d..05fb11e4b620 100644 --- a/server/src/test/java/org/apache/druid/segment/realtime/firehose/ServiceAnnouncingChatHandlerProviderTest.java +++ b/server/src/test/java/org/apache/druid/segment/realtime/ServiceAnnouncingChatHandlerProviderTest.java @@ -17,7 +17,7 @@ * under the License. */ -package org.apache.druid.segment.realtime.firehose; +package org.apache.druid.segment.realtime; import org.apache.druid.curator.discovery.ServiceAnnouncer; import org.apache.druid.server.DruidNode; diff --git a/server/src/test/java/org/apache/druid/segment/realtime/firehose/EventReceiverFirehoseIdleTest.java b/server/src/test/java/org/apache/druid/segment/realtime/firehose/EventReceiverFirehoseIdleTest.java deleted file mode 100644 index 419b29ace7e2..000000000000 --- a/server/src/test/java/org/apache/druid/segment/realtime/firehose/EventReceiverFirehoseIdleTest.java +++ /dev/null @@ -1,136 +0,0 @@ -/* - * Licensed to the Apache Software Foundation (ASF) under one - * or more contributor license agreements. See the NOTICE file - * distributed with this work for additional information - * regarding copyright ownership. The ASF licenses this file - * to you under the Apache License, Version 2.0 (the - * "License"); you may not use this file except in compliance - * with the License. You may obtain a copy of the License at - * - * http://www.apache.org/licenses/LICENSE-2.0 - * - * Unless required by applicable law or agreed to in writing, - * software distributed under the License is distributed on an - * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY - * KIND, either express or implied. See the License for the - * specific language governing permissions and limitations - * under the License. - */ - -package org.apache.druid.segment.realtime.firehose; - -import com.google.common.collect.ImmutableList; -import org.apache.commons.io.IOUtils; -import org.apache.druid.data.input.impl.DimensionsSpec; -import org.apache.druid.data.input.impl.JSONParseSpec; -import org.apache.druid.data.input.impl.MapInputRowParser; -import org.apache.druid.data.input.impl.TimestampSpec; -import org.apache.druid.jackson.DefaultObjectMapper; -import org.apache.druid.server.metrics.EventReceiverFirehoseRegister; -import org.apache.druid.server.security.AllowAllAuthenticator; -import org.apache.druid.server.security.AuthConfig; -import org.apache.druid.server.security.AuthTestUtils; -import org.easymock.EasyMock; -import org.junit.Assert; -import org.junit.Before; -import org.junit.Test; - -import javax.servlet.http.HttpServletRequest; -import java.util.Locale; - -public class EventReceiverFirehoseIdleTest -{ - private static final int CAPACITY = 300; - private static final long MAX_IDLE_TIME = 5_000L; - private static final String SERVICE_NAME = "test_firehose"; - - private final String inputRow = "[{\n" - + " \"timestamp\":123,\n" - + " \"d1\":\"v1\"\n" - + "}]"; - - private EventReceiverFirehoseFactory eventReceiverFirehoseFactory; - private EventReceiverFirehoseFactory.EventReceiverFirehose firehose; - private EventReceiverFirehoseRegister register = new EventReceiverFirehoseRegister(); - private HttpServletRequest req; - - @Before - public void setUp() - { - req = EasyMock.createMock(HttpServletRequest.class); - eventReceiverFirehoseFactory = new EventReceiverFirehoseFactory( - SERVICE_NAME, - CAPACITY, - MAX_IDLE_TIME, - null, - new DefaultObjectMapper(), - new DefaultObjectMapper(), - register, - AuthTestUtils.TEST_AUTHORIZER_MAPPER - ); - firehose = (EventReceiverFirehoseFactory.EventReceiverFirehose) eventReceiverFirehoseFactory.connect( - new MapInputRowParser( - new JSONParseSpec( - new TimestampSpec( - "timestamp", - "auto", - null - ), new DimensionsSpec(DimensionsSpec.getDefaultSchemas(ImmutableList.of("d1"))), - null, - null, - null - ) - ), - null - ); - } - - @Test(timeout = 40_000L) - public void testIdle() throws Exception - { - awaitFirehoseClosed(); - awaitDelayedExecutorThreadTerminated(); - } - - private void awaitFirehoseClosed() throws InterruptedException - { - while (!firehose.isClosed()) { - Thread.sleep(50); - } - } - - private void awaitDelayedExecutorThreadTerminated() throws InterruptedException - { - firehose.getDelayedCloseExecutor().join(); - } - - @Test(timeout = 40_000L) - public void testNotIdle() throws Exception - { - EasyMock.expect(req.getAttribute(AuthConfig.DRUID_AUTHORIZATION_CHECKED)) - .andReturn(null) - .anyTimes(); - EasyMock.expect(req.getAttribute(AuthConfig.DRUID_ALLOW_UNSECURED_PATH)) - .andReturn(null) - .anyTimes(); - EasyMock.expect(req.getAttribute(AuthConfig.DRUID_AUTHENTICATION_RESULT)) - .andReturn(AllowAllAuthenticator.ALLOW_ALL_RESULT) - .anyTimes(); - EasyMock.expect(req.getHeader("X-Firehose-Producer-Id")).andReturn(null).anyTimes(); - EasyMock.expect(req.getContentType()).andReturn("application/json").anyTimes(); - req.setAttribute(AuthConfig.DRUID_AUTHORIZATION_CHECKED, true); - EasyMock.expectLastCall().anyTimes(); - EasyMock.replay(req); - - final int checks = 5; - for (int i = 0; i < checks; i++) { - Assert.assertFalse(firehose.isClosed()); - System.out.printf(Locale.ENGLISH, "Check %d/%d passed\n", i + 1, checks); - firehose.addAll(IOUtils.toInputStream(inputRow), req); - Thread.sleep(3_000L); - } - - awaitFirehoseClosed(); - awaitDelayedExecutorThreadTerminated(); - } -} diff --git a/server/src/test/java/org/apache/druid/segment/realtime/firehose/EventReceiverFirehoseTest.java b/server/src/test/java/org/apache/druid/segment/realtime/firehose/EventReceiverFirehoseTest.java deleted file mode 100644 index 38b16c79cab4..000000000000 --- a/server/src/test/java/org/apache/druid/segment/realtime/firehose/EventReceiverFirehoseTest.java +++ /dev/null @@ -1,442 +0,0 @@ -/* - * Licensed to the Apache Software Foundation (ASF) under one - * or more contributor license agreements. See the NOTICE file - * distributed with this work for additional information - * regarding copyright ownership. The ASF licenses this file - * to you under the Apache License, Version 2.0 (the - * "License"); you may not use this file except in compliance - * with the License. You may obtain a copy of the License at - * - * http://www.apache.org/licenses/LICENSE-2.0 - * - * Unless required by applicable law or agreed to in writing, - * software distributed under the License is distributed on an - * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY - * KIND, either express or implied. See the License for the - * specific language governing permissions and limitations - * under the License. - */ - -package org.apache.druid.segment.realtime.firehose; - -import com.google.common.collect.ImmutableList; -import com.google.common.collect.Iterables; -import org.apache.commons.io.IOUtils; -import org.apache.druid.data.input.impl.DimensionsSpec; -import org.apache.druid.data.input.impl.JSONParseSpec; -import org.apache.druid.data.input.impl.MapInputRowParser; -import org.apache.druid.data.input.impl.TimestampSpec; -import org.apache.druid.jackson.DefaultObjectMapper; -import org.apache.druid.java.util.common.DateTimes; -import org.apache.druid.java.util.common.ISE; -import org.apache.druid.java.util.common.concurrent.Execs; -import org.apache.druid.server.metrics.EventReceiverFirehoseMetric; -import org.apache.druid.server.metrics.EventReceiverFirehoseRegister; -import org.apache.druid.server.security.AllowAllAuthenticator; -import org.apache.druid.server.security.AuthConfig; -import org.apache.druid.server.security.AuthTestUtils; -import org.easymock.EasyMock; -import org.junit.Assert; -import org.junit.Before; -import org.junit.Test; - -import javax.servlet.http.HttpServletRequest; -import javax.ws.rs.core.Response; -import java.io.IOException; -import java.io.InputStream; -import java.nio.charset.StandardCharsets; -import java.util.Map; -import java.util.concurrent.Callable; -import java.util.concurrent.ExecutionException; -import java.util.concurrent.ExecutorService; -import java.util.concurrent.Future; -import java.util.concurrent.TimeUnit; -import java.util.concurrent.TimeoutException; - -public class EventReceiverFirehoseTest -{ - private static final int CAPACITY = 300; - private static final int NUM_EVENTS = 100; - private static final long MAX_IDLE_TIME_MILLIS = TimeUnit.SECONDS.toMillis(20); - private static final String SERVICE_NAME = "test_firehose"; - - private final String inputRow = "[{\n" - + " \"timestamp\":123,\n" - + " \"d1\":\"v1\"\n" - + "}]"; - - private EventReceiverFirehoseFactory eventReceiverFirehoseFactory; - private EventReceiverFirehoseFactory.EventReceiverFirehose firehose; - private EventReceiverFirehoseRegister register = new EventReceiverFirehoseRegister(); - private HttpServletRequest req; - - @Before - public void setUp() - { - req = EasyMock.createMock(HttpServletRequest.class); - eventReceiverFirehoseFactory = new EventReceiverFirehoseFactory( - SERVICE_NAME, - CAPACITY, - MAX_IDLE_TIME_MILLIS, - null, - new DefaultObjectMapper(), - new DefaultObjectMapper(), - register, - AuthTestUtils.TEST_AUTHORIZER_MAPPER - ); - firehose = (EventReceiverFirehoseFactory.EventReceiverFirehose) eventReceiverFirehoseFactory.connect( - new MapInputRowParser( - new JSONParseSpec( - new TimestampSpec( - "timestamp", - "auto", - null - ), new DimensionsSpec(DimensionsSpec.getDefaultSchemas(ImmutableList.of("d1"))), - null, - null, - null - ) - ), - null - ); - } - - @Test(timeout = 60_000L) - public void testSingleThread() throws IOException, InterruptedException - { - for (int i = 0; i < NUM_EVENTS; ++i) { - setUpRequestExpectations(null, null); - final InputStream inputStream = IOUtils.toInputStream(inputRow, StandardCharsets.UTF_8); - firehose.addAll(inputStream, req); - Assert.assertEquals(i + 1, firehose.getCurrentBufferSize()); - inputStream.close(); - } - - EasyMock.verify(req); - - final Iterable> metrics = register.getMetrics(); - Assert.assertEquals(1, Iterables.size(metrics)); - - final Map.Entry entry = Iterables.getLast(metrics); - Assert.assertEquals(SERVICE_NAME, entry.getKey()); - Assert.assertEquals(CAPACITY, entry.getValue().getCapacity()); - Assert.assertEquals(CAPACITY, firehose.getCapacity()); - Assert.assertEquals(NUM_EVENTS, entry.getValue().getCurrentBufferSize()); - Assert.assertEquals(NUM_EVENTS, firehose.getCurrentBufferSize()); - - for (int i = NUM_EVENTS - 1; i >= 0; --i) { - Assert.assertTrue(firehose.hasMore()); - Assert.assertNotNull(firehose.nextRow()); - Assert.assertEquals(i, firehose.getCurrentBufferSize()); - } - - Assert.assertEquals(CAPACITY, entry.getValue().getCapacity()); - Assert.assertEquals(CAPACITY, firehose.getCapacity()); - Assert.assertEquals(0, entry.getValue().getCurrentBufferSize()); - Assert.assertEquals(0, firehose.getCurrentBufferSize()); - - firehose.close(); - Assert.assertFalse(firehose.hasMore()); - Assert.assertEquals(0, Iterables.size(register.getMetrics())); - - awaitDelayedExecutorThreadTerminated(); - } - - @Test(timeout = 60_000L) - public void testMultipleThreads() throws InterruptedException, IOException, TimeoutException, ExecutionException - { - EasyMock.expect(req.getAttribute(AuthConfig.DRUID_AUTHORIZATION_CHECKED)) - .andReturn(null) - .anyTimes(); - EasyMock.expect(req.getAttribute(AuthConfig.DRUID_ALLOW_UNSECURED_PATH)).andReturn(null).anyTimes(); - EasyMock.expect(req.getAttribute(AuthConfig.DRUID_AUTHENTICATION_RESULT)) - .andReturn(AllowAllAuthenticator.ALLOW_ALL_RESULT) - .anyTimes(); - req.setAttribute(AuthConfig.DRUID_AUTHORIZATION_CHECKED, true); - EasyMock.expectLastCall().anyTimes(); - - EasyMock.expect(req.getContentType()).andReturn("application/json").times(2 * NUM_EVENTS); - EasyMock.expect(req.getHeader("X-Firehose-Producer-Id")).andReturn(null).times(2 * NUM_EVENTS); - EasyMock.replay(req); - - final ExecutorService executorService = Execs.singleThreaded("single_thread"); - final Future future = executorService.submit( - new Callable() - { - @Override - public Boolean call() throws Exception - { - for (int i = 0; i < NUM_EVENTS; ++i) { - final InputStream inputStream = IOUtils.toInputStream(inputRow, StandardCharsets.UTF_8); - firehose.addAll(inputStream, req); - inputStream.close(); - } - return true; - } - } - ); - - for (int i = 0; i < NUM_EVENTS; ++i) { - final InputStream inputStream = IOUtils.toInputStream(inputRow, StandardCharsets.UTF_8); - firehose.addAll(inputStream, req); - inputStream.close(); - } - - future.get(10, TimeUnit.SECONDS); - - EasyMock.verify(req); - - final Iterable> metrics = register.getMetrics(); - Assert.assertEquals(1, Iterables.size(metrics)); - - final Map.Entry entry = Iterables.getLast(metrics); - - Assert.assertEquals(SERVICE_NAME, entry.getKey()); - Assert.assertEquals(CAPACITY, entry.getValue().getCapacity()); - Assert.assertEquals(CAPACITY, firehose.getCapacity()); - Assert.assertEquals(2 * NUM_EVENTS, entry.getValue().getCurrentBufferSize()); - Assert.assertEquals(2 * NUM_EVENTS, firehose.getCurrentBufferSize()); - - for (int i = 2 * NUM_EVENTS - 1; i >= 0; --i) { - Assert.assertTrue(firehose.hasMore()); - Assert.assertNotNull(firehose.nextRow()); - Assert.assertEquals(i, firehose.getCurrentBufferSize()); - } - - Assert.assertEquals(CAPACITY, entry.getValue().getCapacity()); - Assert.assertEquals(CAPACITY, firehose.getCapacity()); - Assert.assertEquals(0, entry.getValue().getCurrentBufferSize()); - Assert.assertEquals(0, firehose.getCurrentBufferSize()); - - firehose.close(); - Assert.assertFalse(firehose.hasMore()); - Assert.assertEquals(0, Iterables.size(register.getMetrics())); - - awaitDelayedExecutorThreadTerminated(); - - executorService.shutdownNow(); - } - - @Test(expected = ISE.class) - public void testDuplicateRegistering() - { - EventReceiverFirehoseFactory eventReceiverFirehoseFactory2 = new EventReceiverFirehoseFactory( - SERVICE_NAME, - CAPACITY, - MAX_IDLE_TIME_MILLIS, - null, - new DefaultObjectMapper(), - new DefaultObjectMapper(), - register, - AuthTestUtils.TEST_AUTHORIZER_MAPPER - ); - EventReceiverFirehoseFactory.EventReceiverFirehose firehose2 = - (EventReceiverFirehoseFactory.EventReceiverFirehose) eventReceiverFirehoseFactory2 - .connect( - new MapInputRowParser( - new JSONParseSpec( - new TimestampSpec( - "timestamp", - "auto", - null - ), new DimensionsSpec(DimensionsSpec.getDefaultSchemas(ImmutableList.of("d1"))), - null, - null, - null - ) - ), - null - ); - } - - @Test(timeout = 60_000L) - public void testShutdownWithPrevTime() throws Exception - { - EasyMock.expect(req.getAttribute(AuthConfig.DRUID_AUTHORIZATION_CHECKED)) - .andReturn(null) - .anyTimes(); - EasyMock.expect(req.getAttribute(AuthConfig.DRUID_ALLOW_UNSECURED_PATH)).andReturn(null).anyTimes(); - EasyMock.expect(req.getAttribute(AuthConfig.DRUID_AUTHENTICATION_RESULT)) - .andReturn(AllowAllAuthenticator.ALLOW_ALL_RESULT) - .anyTimes(); - req.setAttribute(AuthConfig.DRUID_AUTHORIZATION_CHECKED, true); - EasyMock.expectLastCall().anyTimes(); - EasyMock.replay(req); - - firehose.shutdown(DateTimes.nowUtc().minusMinutes(2).toString(), req); - awaitFirehoseClosed(); - awaitDelayedExecutorThreadTerminated(); - } - - private void awaitFirehoseClosed() throws InterruptedException - { - while (!firehose.isClosed()) { - Thread.sleep(50); - } - } - - private void awaitDelayedExecutorThreadTerminated() throws InterruptedException - { - firehose.getDelayedCloseExecutor().join(); - } - - @Test(timeout = 60_000L) - public void testShutdown() throws Exception - { - EasyMock.expect(req.getAttribute(AuthConfig.DRUID_AUTHORIZATION_CHECKED)) - .andReturn(null) - .anyTimes(); - EasyMock.expect(req.getAttribute(AuthConfig.DRUID_ALLOW_UNSECURED_PATH)).andReturn(null).anyTimes(); - EasyMock.expect(req.getAttribute(AuthConfig.DRUID_AUTHENTICATION_RESULT)) - .andReturn(AllowAllAuthenticator.ALLOW_ALL_RESULT) - .anyTimes(); - req.setAttribute(AuthConfig.DRUID_AUTHORIZATION_CHECKED, true); - EasyMock.expectLastCall().anyTimes(); - EasyMock.replay(req); - - firehose.shutdown(DateTimes.nowUtc().plusMillis(100).toString(), req); - awaitFirehoseClosed(); - awaitDelayedExecutorThreadTerminated(); - } - - @Test - public void testProducerSequence() throws IOException - { - for (int i = 0; i < NUM_EVENTS; ++i) { - setUpRequestExpectations("producer", String.valueOf(i)); - - final InputStream inputStream = IOUtils.toInputStream(inputRow, StandardCharsets.UTF_8); - firehose.addAll(inputStream, req); - Assert.assertEquals(i + 1, firehose.getCurrentBufferSize()); - inputStream.close(); - } - - EasyMock.verify(req); - - final Iterable> metrics = register.getMetrics(); - Assert.assertEquals(1, Iterables.size(metrics)); - - final Map.Entry entry = Iterables.getLast(metrics); - Assert.assertEquals(SERVICE_NAME, entry.getKey()); - Assert.assertEquals(CAPACITY, entry.getValue().getCapacity()); - Assert.assertEquals(CAPACITY, firehose.getCapacity()); - Assert.assertEquals(NUM_EVENTS, entry.getValue().getCurrentBufferSize()); - Assert.assertEquals(NUM_EVENTS, firehose.getCurrentBufferSize()); - - for (int i = NUM_EVENTS - 1; i >= 0; --i) { - Assert.assertTrue(firehose.hasMore()); - Assert.assertNotNull(firehose.nextRow()); - Assert.assertEquals(i, firehose.getCurrentBufferSize()); - } - - Assert.assertEquals(CAPACITY, entry.getValue().getCapacity()); - Assert.assertEquals(CAPACITY, firehose.getCapacity()); - Assert.assertEquals(0, entry.getValue().getCurrentBufferSize()); - Assert.assertEquals(0, firehose.getCurrentBufferSize()); - - firehose.close(); - Assert.assertFalse(firehose.hasMore()); - Assert.assertEquals(0, Iterables.size(register.getMetrics())); - } - - @Test - public void testLowProducerSequence() throws IOException - { - for (int i = 0; i < NUM_EVENTS; ++i) { - setUpRequestExpectations("producer", "1"); - - final InputStream inputStream = IOUtils.toInputStream(inputRow, StandardCharsets.UTF_8); - final Response response = firehose.addAll(inputStream, req); - Assert.assertEquals(Response.Status.OK.getStatusCode(), response.getStatus()); - Assert.assertEquals(1, firehose.getCurrentBufferSize()); - inputStream.close(); - } - - EasyMock.verify(req); - - firehose.close(); - } - - @Test - public void testMissingProducerSequence() throws IOException - { - setUpRequestExpectations("producer", null); - - final InputStream inputStream = IOUtils.toInputStream(inputRow, StandardCharsets.UTF_8); - final Response response = firehose.addAll(inputStream, req); - - Assert.assertEquals(Response.Status.BAD_REQUEST.getStatusCode(), response.getStatus()); - - inputStream.close(); - - EasyMock.verify(req); - - firehose.close(); - } - - @Test - public void testTooManyProducerIds() throws IOException - { - for (int i = 0; i < EventReceiverFirehoseFactory.MAX_FIREHOSE_PRODUCERS - 1; i++) { - setUpRequestExpectations("producer-" + i, "0"); - - final InputStream inputStream = IOUtils.toInputStream(inputRow, StandardCharsets.UTF_8); - final Response response = firehose.addAll(inputStream, req); - Assert.assertEquals(Response.Status.OK.getStatusCode(), response.getStatus()); - inputStream.close(); - Assert.assertTrue(firehose.hasMore()); - Assert.assertNotNull(firehose.nextRow()); - } - - setUpRequestExpectations("toomany", "0"); - - final InputStream inputStream = IOUtils.toInputStream(inputRow, StandardCharsets.UTF_8); - final Response response = firehose.addAll(inputStream, req); - Assert.assertEquals(Response.Status.FORBIDDEN.getStatusCode(), response.getStatus()); - inputStream.close(); - - EasyMock.verify(req); - - firehose.close(); - } - - @Test - public void testNaNProducerSequence() throws IOException - { - setUpRequestExpectations("producer", "foo"); - - final InputStream inputStream = IOUtils.toInputStream(inputRow, StandardCharsets.UTF_8); - final Response response = firehose.addAll(inputStream, req); - - Assert.assertEquals(Response.Status.BAD_REQUEST.getStatusCode(), response.getStatus()); - - inputStream.close(); - - EasyMock.verify(req); - - firehose.close(); - } - - private void setUpRequestExpectations(String producerId, String producerSequenceValue) - { - EasyMock.reset(req); - EasyMock.expect(req.getAttribute(AuthConfig.DRUID_AUTHORIZATION_CHECKED)) - .andReturn(null) - .anyTimes(); - EasyMock.expect(req.getAttribute(AuthConfig.DRUID_ALLOW_UNSECURED_PATH)).andReturn(null).anyTimes(); - EasyMock.expect(req.getAttribute(AuthConfig.DRUID_AUTHENTICATION_RESULT)) - .andReturn(AllowAllAuthenticator.ALLOW_ALL_RESULT) - .anyTimes(); - req.setAttribute(AuthConfig.DRUID_AUTHORIZATION_CHECKED, true); - EasyMock.expectLastCall().anyTimes(); - - EasyMock.expect(req.getContentType()).andReturn("application/json"); - EasyMock.expect(req.getHeader("X-Firehose-Producer-Id")).andReturn(producerId); - - if (producerId != null) { - EasyMock.expect(req.getHeader("X-Firehose-Producer-Seq")).andReturn(producerSequenceValue); - } - - EasyMock.replay(req); - } -} diff --git a/services/src/main/java/org/apache/druid/cli/CliIndexer.java b/services/src/main/java/org/apache/druid/cli/CliIndexer.java index 312b6f6b05a3..5decef93018c 100644 --- a/services/src/main/java/org/apache/druid/cli/CliIndexer.java +++ b/services/src/main/java/org/apache/druid/cli/CliIndexer.java @@ -36,7 +36,6 @@ import org.apache.druid.discovery.WorkerNodeService; import org.apache.druid.guice.DruidProcessingModule; import org.apache.druid.guice.IndexerServiceModule; -import org.apache.druid.guice.IndexingServiceFirehoseModule; import org.apache.druid.guice.IndexingServiceInputSourceModule; import org.apache.druid.guice.IndexingServiceModuleHelper; import org.apache.druid.guice.IndexingServiceTaskLogsModule; @@ -238,7 +237,6 @@ public DataNodeService getDataNodeService(DruidServerConfig serverConfig) } }, new ShuffleModule(), - new IndexingServiceFirehoseModule(), new IndexingServiceInputSourceModule(), new IndexingServiceTaskLogsModule(), new IndexingServiceTuningConfigModule(), diff --git a/services/src/main/java/org/apache/druid/cli/CliMiddleManager.java b/services/src/main/java/org/apache/druid/cli/CliMiddleManager.java index 37b6501bf87f..2e542bc69745 100644 --- a/services/src/main/java/org/apache/druid/cli/CliMiddleManager.java +++ b/services/src/main/java/org/apache/druid/cli/CliMiddleManager.java @@ -36,7 +36,6 @@ import org.apache.druid.curator.ZkEnablementConfig; import org.apache.druid.discovery.NodeRole; import org.apache.druid.discovery.WorkerNodeService; -import org.apache.druid.guice.IndexingServiceFirehoseModule; import org.apache.druid.guice.IndexingServiceInputSourceModule; import org.apache.druid.guice.IndexingServiceModuleHelper; import org.apache.druid.guice.IndexingServiceTaskLogsModule; @@ -73,10 +72,10 @@ import org.apache.druid.query.DruidMetrics; import org.apache.druid.query.lookup.LookupSerdeModule; import org.apache.druid.segment.incremental.RowIngestionMetersFactory; +import org.apache.druid.segment.realtime.ChatHandlerProvider; +import org.apache.druid.segment.realtime.NoopChatHandlerProvider; import org.apache.druid.segment.realtime.appenderator.AppenderatorsManager; import org.apache.druid.segment.realtime.appenderator.DummyForInjectionAppenderatorsManager; -import org.apache.druid.segment.realtime.firehose.ChatHandlerProvider; -import org.apache.druid.segment.realtime.firehose.NoopChatHandlerProvider; import org.apache.druid.server.DruidNode; import org.apache.druid.server.http.SelfDiscoveryResource; import org.apache.druid.server.initialization.jetty.JettyServerInitializer; @@ -242,7 +241,6 @@ public WorkerNodeService getWorkerNodeService(WorkerConfig workerConfig) } }, new ShuffleModule(), - new IndexingServiceFirehoseModule(), new IndexingServiceInputSourceModule(), new IndexingServiceTaskLogsModule(), new IndexingServiceTuningConfigModule(), diff --git a/services/src/main/java/org/apache/druid/cli/CliOverlord.java b/services/src/main/java/org/apache/druid/cli/CliOverlord.java index 883b7828822a..636d775f030f 100644 --- a/services/src/main/java/org/apache/druid/cli/CliOverlord.java +++ b/services/src/main/java/org/apache/druid/cli/CliOverlord.java @@ -39,7 +39,6 @@ import com.google.inject.util.Providers; import org.apache.druid.client.indexing.IndexingService; import org.apache.druid.discovery.NodeRole; -import org.apache.druid.guice.IndexingServiceFirehoseModule; import org.apache.druid.guice.IndexingServiceInputSourceModule; import org.apache.druid.guice.IndexingServiceModuleHelper; import org.apache.druid.guice.IndexingServiceTaskLogsModule; @@ -108,10 +107,10 @@ import org.apache.druid.query.lookup.LookupSerdeModule; import org.apache.druid.segment.incremental.RowIngestionMetersFactory; import org.apache.druid.segment.metadata.CentralizedDatasourceSchemaConfig; +import org.apache.druid.segment.realtime.ChatHandlerProvider; +import org.apache.druid.segment.realtime.NoopChatHandlerProvider; import org.apache.druid.segment.realtime.appenderator.AppenderatorsManager; import org.apache.druid.segment.realtime.appenderator.DummyForInjectionAppenderatorsManager; -import org.apache.druid.segment.realtime.firehose.ChatHandlerProvider; -import org.apache.druid.segment.realtime.firehose.NoopChatHandlerProvider; import org.apache.druid.server.coordinator.CoordinatorOverlordServiceConfig; import org.apache.druid.server.http.RedirectFilter; import org.apache.druid.server.http.RedirectInfo; @@ -428,7 +427,6 @@ private void configureOverlordHelpers(Binder binder) .to(TaskLogAutoCleaner.class); } }, - new IndexingServiceFirehoseModule(), new IndexingServiceInputSourceModule(), new IndexingServiceTaskLogsModule(), new IndexingServiceTuningConfigModule(), diff --git a/services/src/main/java/org/apache/druid/cli/CliPeon.java b/services/src/main/java/org/apache/druid/cli/CliPeon.java index 5edb83d0a3b5..eb572850cda2 100644 --- a/services/src/main/java/org/apache/druid/cli/CliPeon.java +++ b/services/src/main/java/org/apache/druid/cli/CliPeon.java @@ -48,7 +48,6 @@ import org.apache.druid.guice.Binders; import org.apache.druid.guice.CacheModule; import org.apache.druid.guice.DruidProcessingModule; -import org.apache.druid.guice.IndexingServiceFirehoseModule; import org.apache.druid.guice.IndexingServiceInputSourceModule; import org.apache.druid.guice.IndexingServiceTaskLogsModule; import org.apache.druid.guice.IndexingServiceTuningConfigModule; @@ -116,11 +115,11 @@ import org.apache.druid.segment.loading.OmniDataSegmentMover; import org.apache.druid.segment.loading.StorageLocation; import org.apache.druid.segment.metadata.CentralizedDatasourceSchemaConfig; +import org.apache.druid.segment.realtime.ChatHandlerProvider; +import org.apache.druid.segment.realtime.NoopChatHandlerProvider; +import org.apache.druid.segment.realtime.ServiceAnnouncingChatHandlerProvider; import org.apache.druid.segment.realtime.appenderator.AppenderatorsManager; import org.apache.druid.segment.realtime.appenderator.PeonAppenderatorsManager; -import org.apache.druid.segment.realtime.firehose.ChatHandlerProvider; -import org.apache.druid.segment.realtime.firehose.NoopChatHandlerProvider; -import org.apache.druid.segment.realtime.firehose.ServiceAnnouncingChatHandlerProvider; import org.apache.druid.server.DruidNode; import org.apache.druid.server.ResponseContextConfig; import org.apache.druid.server.SegmentManager; @@ -343,7 +342,6 @@ public LookupLoadingSpec getLookupsToLoad(final Task task) } }, new QueryablePeonModule(), - new IndexingServiceFirehoseModule(), new IndexingServiceInputSourceModule(), new IndexingServiceTuningConfigModule(), new InputSourceModule(), diff --git a/services/src/main/java/org/apache/druid/cli/validate/DruidJsonValidator.java b/services/src/main/java/org/apache/druid/cli/validate/DruidJsonValidator.java index 0518c7259e63..50684cee0e87 100644 --- a/services/src/main/java/org/apache/druid/cli/validate/DruidJsonValidator.java +++ b/services/src/main/java/org/apache/druid/cli/validate/DruidJsonValidator.java @@ -40,8 +40,6 @@ import org.apache.druid.data.input.impl.StringInputRowParser; import org.apache.druid.guice.DruidProcessingModule; import org.apache.druid.guice.ExtensionsLoader; -import org.apache.druid.guice.FirehoseModule; -import org.apache.druid.guice.IndexingServiceFirehoseModule; import org.apache.druid.guice.IndexingServiceInputSourceModule; import org.apache.druid.guice.LocalDataStorageDruidModule; import org.apache.druid.guice.QueryRunnerFactoryModule; @@ -130,9 +128,7 @@ public void run() Iterables.concat( extnLoader.getModules(), Arrays.asList( - new FirehoseModule(), new IndexingHadoopModule(), - new IndexingServiceFirehoseModule(), new IndexingServiceInputSourceModule(), new LocalDataStorageDruidModule() ) diff --git a/services/src/test/java/org/apache/druid/cli/validate/DruidJsonValidatorTest.java b/services/src/test/java/org/apache/druid/cli/validate/DruidJsonValidatorTest.java index b843465147eb..b617b7e6b877 100644 --- a/services/src/test/java/org/apache/druid/cli/validate/DruidJsonValidatorTest.java +++ b/services/src/test/java/org/apache/druid/cli/validate/DruidJsonValidatorTest.java @@ -142,7 +142,6 @@ public void testTaskValidator() throws Exception jsonMapper ), new IndexTask.IndexIOConfig( - null, new LocalInputSource(new File("lol"), "rofl"), new JsonInputFormat(null, null, null, null, null), false, diff --git a/website/.spelling b/website/.spelling index 40b1b906a12b..4636ea61c901 100644 --- a/website/.spelling +++ b/website/.spelling @@ -98,7 +98,6 @@ EMRFS ETL Elasticsearch Enums -FirehoseFactory FIRST_VALUE FlattenSpec Float.NEGATIVE_INFINITY @@ -113,7 +112,6 @@ GUIs GroupBy Guice HDFS -HDFSFirehose HLL HashSet Homebrew @@ -229,7 +227,6 @@ SSDs SSL Samza Splunk -SqlFirehose SqlParameter SslContextFactory StatsD @@ -510,6 +507,7 @@ schemas schemaless searchable secondaryPartitionPruning +seekable seekable-stream servlet setProcessingThreadNames @@ -880,8 +878,6 @@ hadoop3 ambari-metrics metricName trustStore -StaticAzureBlobStoreFirehose -StaticS3Firehose fetchTimeout gz maxCacheCapacityBytes @@ -890,11 +886,9 @@ maxFetchRetry prefetchTriggerBytes shardSpecs sharedAccessStorageToken -StaticCloudFilesFirehose cloudfiles rackspace-cloudfiles-uk rackspace-cloudfiles-us -StaticAzureBlobStoreFirehose gz shardSpecs maxCacheCapacityBytes @@ -1096,7 +1090,6 @@ POD_NAMESPACE ConfigMap PT17S GCS -StaticGoogleBlobStoreFirehose gcs-connector hdfs Aotearoa @@ -1296,7 +1289,6 @@ QueryRunnerFactory segmentmetadataquery SegmentMetadataQuery SegmentMetadataQueryQueryToolChest -StaticS3FirehoseFactory loadSpec multibind pom.xml @@ -1337,10 +1329,8 @@ valueFormat IOConfig compactionTask compactionTasks -ingestSegmentFirehose numShards IngestSegment -IngestSegmentFirehose maxSizes snapshotTime windowPeriod @@ -1402,14 +1392,10 @@ useYarnRMJobStatusFallback workingPath z.example.com 150MB -CombiningFirehose DataSchema DefaultPassword EnvironmentVariablePasswordProvider -HttpFirehose IOConfig -InlineFirehose -LocalFirehose PartitionsSpec PasswordProviders SegmentsSplitHintSpec @@ -1446,12 +1432,9 @@ splitHintSpec taskStatusCheckPeriodMs timeChunk totalNumMergeTasks -StaticS3Firehose prefetchTriggerBytes awaitSegmentAvailabilityTimeoutMillis -LocalFirehose baseDir -HttpFirehose httpAuthenticationUsername DefaultPassword PasswordProviders @@ -1462,8 +1445,6 @@ maxInputSegmentBytesPerTask foldCase sqls connectorConfig -InlineFirehose -CombiningFirehose httpAuthenticationPassword accessKeyId secretAccessKey @@ -1660,8 +1641,6 @@ Temurin 0.14.x 1s Bufferpool -EventReceiverFirehose -EventReceiverFirehoseMonitor Filesystem JVMMonitor jvmVersion @@ -1984,7 +1963,6 @@ GiB Autoscaler APPROX_COUNT_DISTINCT_BUILTIN AvaticaConnectionBalancer -EventReceiverFirehose File.getFreeSpace File.getTotalSpace ForkJoinPool