Commit Graph

14280 Commits

Author SHA1 Message Date
Rushikesh Bankar 4ef4e75c5d
Fix the issue of missing implementation of IndexerTaskCountStatsProvider for peons (#16875)
Bug description:
Peons to fail to start up when `WorkerTaskCountStatsMonitor` is used on MiddleManagers.
This is because MiddleManagers pass on their properties to peons and peons are unable to
find `IndexerTaskCountStatsProvider` as that is bound only for indexer nodes.

Fix:
Check if node is an indexer before trying to get instance of `IndexerTaskCountStatsProvider`.
2024-08-10 14:53:16 +05:30
Vadim Ogievetsky 483a03f26c
Web console: Server context defaults (#16868)
* add server defaults

* null is NULL

* r to d

* add test

* typo
2024-08-09 14:46:59 -07:00
Adithya Chakilam a7dd436a32
Check if supervisor could be idle on startup (#16844)
Fixes #13936 

In cases where a supervisor is idle and the overlord is restarted for some reason, the supervisor would
start spinning tasks again. In clusters where there are many low throughput streams, this would spike
the task count unnecessarily.

This commit compares the latest stream offset with the ones in metadata during the startup of supervisor
and sets it to idle state if they match.
2024-08-09 14:42:48 +05:30
Akshat Jain 3d6cedb25f
Fix IndexOutOfBoundsException for MSQ window function queries with empty RAC (#16865)
* Fix IndexOutOfBoundsException for MSQ window function queries with empty RAC
2024-08-09 11:39:53 +05:30
zachjsh cb09b572e6
Fix Druid table schema resolution when table defined in catalog and has schema manager (#16869)
* SQL syntax error should target USER persona

* * revert change to queryHandler and related tests, based on review comments

* * add test

* Properly handle Druid schema blending with catalog definition and segment metadata

* * add javadocs
2024-08-08 21:21:03 -04:00
Clint Wylie 6cd8c6be22
fix IndexedStringDruidPredicateIndexes to not needlessly lookup index of values (#16860) 2024-08-07 23:29:56 -07:00
Akshat Jain 7f67d26dfa
Reduce logging in RetryableS3OutputStream (#16853)
This PR reduces logging in RetryableS3OutputStream.
2024-08-08 10:42:40 +05:30
Zoltan Haindrich 408702e100
Add ability to run MSQ in Quidem tests (#16798)
* implements some jdbc facade to enable msq usage
* adds an !msqPlan command
* adds more guice usage to testsystem startup
2024-08-08 06:37:06 +02:00
Hardik Bajaj 1cf3f4bebe
Fix Concurrent Task Insertion in pendingCompletionTaskGroups (#16834)
Fix streaming task failures that may arise due to concurrent task insertion in pendingCompletionTaskGroups
2024-08-08 08:37:27 +05:30
aaronm-bi ceed4a0634
Docs: Update list of ingestion types that support concurrent append and replace (#16852) 2024-08-08 08:06:22 +05:30
Vadim Ogievetsky 56c03582cf
support kinesis input format (#16850) 2024-08-07 10:24:28 -07:00
Rishabh Singh c6a7ab005f
Increase query cancellation timeout in the router (#16656)
* Fix build

* Increase query cancellation timeout in router

* Increase cancellation timeout to 5 seconds
2024-08-07 20:29:35 +05:30
Atul Mohan 76ad17fb4c
Add config for http client connect timeout (#16831)
Adds a configuration clientConnectTimeout to our http client config which controls the connection timeout for our http client requests.

It was observed that on busy K8S clusters, the default connect timeout of 500ms is sometimes not enough time to complete syn/acks for a request and in these cases, the requests timeout with the error:
exceptionType=java.net.SocketTimeoutException, exceptionMessage=Connect Timeout
This behavior was mostly observed on the router while forwarding queries to the broker.
Having a slightly higher connect timeout helped resolve these issues.
2024-08-07 19:31:10 +05:30
Sree Charan Manamala 84192b11d7
Benchmark for window functions (#16824) 2024-08-07 11:07:11 +02:00
Sree Charan Manamala 1f6d2c41d2
Update doc for dynamic parameters supporting array (#16660)
Update dynamic parameter docs to provide how it can used to replace an Array
2024-08-07 12:33:37 +05:30
Rishabh Singh 99313e9996
Revised IT to detect backward incompatible change (#16779)
Added a new revised IT group BackwardCompatibilityMain. The idea is to catch potential backward compatibility issues that may arise during rolling upgrade.

This test group runs a docker-compose cluster with Overlord & Coordinator service on the previous druid version.

Following env vars are required in the GHA file .github/workflows/unit-and-integration-tests-unified.yml to run this test

DRUID_PREVIOUS_VERSION -> Previous druid version to test backward incompatibility.
DRUID_PREVIOUS_VERSION_DOWNLOAD_URL -> URL to fetch the tar.
2024-08-07 11:13:35 +05:30
Edgar Melendrez 83cf4dc554
[docs] fixes to sql-scalar.md (#16826)
Co-authored-by: Victoria Lim <vtlim@users.noreply.github.com>
Co-authored-by: Katya Macedo  <38017980+ektravel@users.noreply.github.com>
Co-authored-by: Benedict Jin <asdf2014@apache.org>
Co-authored-by: Katya Macedo <38017980+ektravel@users.noreply.github.com>
2024-08-06 17:12:57 -07:00
zachjsh c324f09108
Kinesis input format docs (#16840)
* SQL syntax error should target USER persona

* * revert change to queryHandler and related tests, based on review comments

* * add test

* Docs for Kinesis input format

* * remove reference to kafka

* * fix spellcheck error

* Apply suggestions from code review

Co-authored-by: 317brian <53799971+317brian@users.noreply.github.com>

---------

Co-authored-by: 317brian <53799971+317brian@users.noreply.github.com>
2024-08-06 18:53:10 -04:00
Gian Merlino eaa09937bc
SuperSorter: direct merging, increased parallelism. (#16775)
Two performance enhancements:

1) Direct merging of input frames to output channels, without any
   temporary files, if all input frames fit in memory.

2) When doing multi-level merging (now called "external mode"),
   improve parallelism by boosting up the number of mergers in the
   penultimate level.

To support direct merging, FrameChannelMerger is enhanced such that the
output partition min/max values are used to filter input frames. This
is necessary because all direct mergers read all input frames, but only
rows corresponding to a single output partition.
2024-08-06 15:00:39 -07:00
Edgar Melendrez ebea34a814
[Docs] Batch06: starting string functions (#16838)
* batch06, starting string functions

* addind space after Syntax

* quick change

* correcting spelling

* Update docs/querying/sql-functions.md

* Update sql-functions.md

* applying suggestions

* Update docs/querying/sql-functions.md

* Update docs/querying/sql-functions.md

---------

Co-authored-by: Benedict Jin <asdf2014@apache.org>
Co-authored-by: Charles Smith <techdocsmith@gmail.com>
2024-08-06 11:32:26 -07:00
Adarsh Sanjeev 739068469c
General Druid refactors (#16708)
Some general refactors across Druid.

    Switch to DruidExceptions
    Add javadocs
    Fix a bug in IntArrayColumns
    Add a class for LongArrayColumns
    Remove wireTransferable since it would never be called
    Refactor DictionaryWriter to return the index written as a return value from write.
2024-08-06 11:47:08 -05:00
Adarsh Sanjeev 2b81c18fd7
Refactor SemanticCreator (#16700)
Refactors the SemanticCreator annotation.

    Moves the interface to the semantic package.
    Create a SemanticUtils to hold logic for storing semantic maps.
    Add FrameMaker interface.
2024-08-06 11:29:38 -05:00
Vishesh Garg 593c3b2150
Do not support non-idempotent aggregator in MSQ compaction (#16846)
This PR adds checks for verification of DataSourceCompactionConfig and CompactionTask with msq engine to ensure:

each aggregator in metricsSpec is idempotent
metricsSpec is non-null when rollup is set to true
Unit tests and existing compaction ITs have been updated accordingly.
2024-08-06 20:58:08 +05:30
Kashif Faraz aa49be61ea
Do not create ZK paths if not needed (#16816)
Background:
ZK-based segment loading has been completely disabled in #15705 .
ZK `servedSegmentsPath` has been deprecated since Druid 0.7.1, #1182 .
This legacy path has been replaced by the `liveSegmentsPath` and is not used in the code anymore.

Changes:
- Never create ZK loadQueuePath as it is never used.
- Never create ZK servedSegmentsPath as it is never used.
- Do not create ZK liveSegmentsPath if announcement on ZK is disabled
- Fix up tests
2024-08-06 19:29:13 +05:30
Gian Merlino de40d81b29
SQL: Add ProjectableFilterableTable to SegmentsTable. (#16841)
* SQL: Add ProjectableFilterableTable to SegmentsTable.

This allows us to skip serialization of expensive fields such as
shard_spec, dimensions, metrics, and last_compaction_state, if those
fields are not actually being queried.

* Restructure logic to avoid unnecessary toString() as well.
2024-08-06 06:40:21 -07:00
Akshat Jain c3aa033e14
MSQ window functions: Fix query correctness issues when using multiple workers (#16804)
This PR fixes query correctness issues for MSQ window functions when using more than 1 worker (that is, maxNumTasks > 2).

Currently, we were keeping the shuffle spec of the previous stage when we didn't have any partition columns for window stage. This PR changes it to override the shuffle spec of the previous stage to MixShuffleSpec (if we have a window function with empty over clause) so that the window stage gets a single partition to work on.

A test has been added for a query which returned incorrect results prior to this change when using more than 1 workers.
2024-08-06 16:11:18 +05:30
Sree Charan Manamala ed6b547481
Handle default bounds correctly in WINDOW clause (#16833)
When a window is defined as WINDOW W AS <DEF> and using a syntax of (PARTITION BY col1 ORDER BY col2 ROWS x PRECEDING), we would need to default the other bound to CURRENT ROW

We already have implemented this earlier, but when defined as WINDOW W AS <DEF>, Calcite takes a different route to validate the window.
2024-08-06 09:58:44 +02:00
Vadim Ogievetsky aeace28ccb
Web console: Add columnMapping information to the Explain dialog (#16598)
* Add columnMapping information in the Explain dialog

* use arrow char
2024-08-05 13:21:51 -07:00
Alberic Liu 461727de12
Fix Druid Console cannot open submit supervisor dialog (#16736) 2024-08-05 10:44:11 -07:00
Zoltan Haindrich 26e3c44f4b
Quidem record (#16624)
* enables to launch a fake broker based on test resources (druidtest uri)
* could record queries into new testfiles during usage
* instead of re-purpose Calcite's Hook migrates to use DruidHook which we can add further keys
* added a quidem-ut module which could be the place for tests which could iteract with modules/etc
2024-08-05 14:58:32 +02:00
Akshat Jain 08f9ec1cae
Memoize the redundant calls to overlord in sql statements endpoint (#16839) 2024-08-05 16:52:56 +05:30
Rushikesh Bankar c8323d1a7c
Add indexer task success and failure metrics (#16829)
This PR adds indexer-level task metrics-

"indexer/task/failed/count"
"indexer/task/success/count"

the current "worker/task/completed/count" metric shows all the tasks completed irrespective of success or failure status so these metrics would help us get more visibility into the status of the completed tasks
2024-08-05 16:21:27 +05:30
Laksh Singla c84e689eb8
Don't use ComplexMetricExtractor to fetch the class of the object in field readers (#16825)
This patch fixes queries like `SELECT COUNT(DISTINCT json_col) FROM foo`
2024-08-05 14:13:56 +05:30
Laksh Singla 0411c4e67e
Add metrics for number of rows/bytes materialized while running subqueries (#16835)
subquery/rows and subquery/bytes metrics have been added, which indicate the size of the results materialized on the heap.
2024-08-05 14:13:20 +05:30
Sree Charan Manamala c7eacd079e
fallback SQL IN filter to expression filter when VirtualColumnRegistry is null (#16836) 2024-08-05 11:27:51 +05:30
Abhishek Radhakrishnan 31b43753fb
Add `druid.indexing.formats.stringMultiValueHandlingMode` system config (#16822)
This patch introduces an optional cluster configuration, druid.indexing.formats.stringMultiValueHandlingMode, allowing operators to override the default mode SORTED_SET for string dimensions. The possible values for the config are SORTED_SET, SORTED_ARRAY, or ARRAY (SORTED_SET is the default). Case insensitive values are allowed.
While this cluster property allows users to manage the multi-value handling mode for string dimension types, it's recommended to migrate to using real array types instead of MVDs.
 
This fixes a long-standing issue where compaction will honor the configured cluster wide property instead of rewriting it as the default SORTED_ARRAY always, even if the data was originally ingested with ARRAY or SORTED_SET.
2024-08-03 10:23:44 -07:00
Kashif Faraz 9dc2569f22
Track and emit segment loading rate for HttpLoadQueuePeon on Coordinator (#16691)
Design:
The loading rate is computed as a moving average of at least the last 10 GiB of successful segment loads.
To account for multiple loading threads on a server, we use the concept of a batch to track load times.
A batch is a set of segments added by the coordinator to the load queue of a server in one go.

Computation:
batchDurationMillis = t(load queue becomes empty) - t(first load request in batch is sent to server)
batchBytes = total bytes successfully loaded in batch
avg loading rate in batch (kbps) = (8 * batchBytes) / batchDurationMillis
overall avg loading rate (kbps) = (8 * sumOverWindow(batchBytes)) / sumOverWindow(batchDurationMillis)

Changes:
- Add `LoadingRateTracker` which computes a moving average load rate based on
the last few GBs of successful segment loads.
- Emit metric `segment/loading/rateKbps` from the Coordinator. In the future, we may
also consider emitting this metric from the historicals themselves.
- Add `expectedLoadTimeMillis` to response of API `/druid/coordinator/v1/loadQueue?simple`
2024-08-03 13:14:21 +05:30
Abhishek Radhakrishnan fe6772a101
Rename test builder `MSQTester.setExpectedSegment` (#16837)
* Rename setExpectedSegment to setExpectedSegments in MSQTestBase.

* Add expected segments for max num segments test cases.
2024-08-02 10:01:55 -07:00
zachjsh 9b731e8f0a
Kinesis Input Format for timestamp, and payload parsing (#16813)
* SQL syntax error should target USER persona

* * revert change to queryHandler and related tests, based on review comments

* * add test

* Introduce KinesisRecordEntity to support Kinesis headers in InputFormats

* * add kinesisInputFormat and Reader, and tests

* * bind KinesisInputFormat class to module

* * improve test coverage

* * remove references to kafka

* * resolve review comments

* * remove comment

* * fix grammer of comment

* * fix comment again

* * fix comment again

* * more review comments

* * add partitionKey

* * add check for same timestamp and partitionKey column name

* * fix intellij inspection
2024-08-02 08:48:44 -04:00
Akshat Jain 63ba5a4113
Fix issues with fetching task reports in SQL statements endpoint for middlemanager (#16832) 2024-08-01 23:37:15 -04:00
Vadim Ogievetsky 8c170f7d0e
Web console: use stages, counters, and warnings from the new detailed status API (#16809)
* stages and counters can be seen on the status reponse

* warnings are exposed also

* mark as msq when attached

* update snapshots

* download CSV/TSV null as empty cell
2024-08-01 02:30:30 -07:00
Akshat Jain bb4d6cc001
Add task report fields in response of SQL statements endpoint (#16808)
If the optional query parameter detail is supplied, then the response also includes the following:

 * A stages object that summarizes information about the different stages being used for query execution, such as stage number, phase, start time, duration, input and output information, processing methods, and partitioning.
* A counters object that provides details on the rows, bytes, and files processed at various stages for each worker across different channels, along with sort progress.
* A warnings object that provides details about any warnings.
2024-08-01 10:26:04 +05:30
Gian Merlino 01f6cfcbf5
MSQ worker: Support in-memory shuffles. (#16790)
* MSQ worker: Support in-memory shuffles.

This patch is a follow-up to #16168, adding worker-side support for
in-memory shuffles. Changes include:

1) Worker-side code now respects the same context parameter "maxConcurrentStages"
   that was added to the controller in #16168. The parameter remains undocumented
   for now, to give us a chance to more fully develop and test this functionality.

1) WorkerImpl is broken up into WorkerImpl, RunWorkOrder, and RunWorkOrderListener
   to improve readability.

2) WorkerImpl has a new StageOutputHolder + StageOutputReader concept, which
   abstract over memory-based or file-based stage results.

3) RunWorkOrder is updated to create in-memory stage output channels when
   instructed to.

4) ControllerResource is updated to add /doneReadingInput/, so the controller
   can tell when workers that sort, but do not gather statistics, are done reading
   their inputs.

5) WorkerMemoryParameters is updated to consider maxConcurrentStages.

Additionally, WorkerChatHandler is split into WorkerResource, so as to match
ControllerChatHandler and ControllerResource.

* Updates for static checks, test coverage.

* Fixes.

* Remove exception.

* Changes from review.

* Address static check.

* Changes from review.

* Improvements to docs and method names.

* Update comments, add test.

* Additional javadocs.

* Fix throws.

* Fix worker stopping in tests.

* Fix stuck test.
2024-07-30 18:41:24 -07:00
Edgar Melendrez 3bb6d40285
[docs] batch 5 updating functions (#16812)
* batch 5

* Update docs/querying/sql-functions.md

* applying suggestions

---------

Co-authored-by: Benedict Jin <asdf2014@apache.org>
2024-07-30 17:30:01 -07:00
Edgar Melendrez 85a8a1d805
[Docs]Batch04 - Bitwise numeric functions (#16805)
* Batch04 - Bitwise numeric functions

* Batch04 - Bitwise numeric functions

* minor fixes

* rewording bitwise_shift functions

* rewording bitwise_shift functions

* Update docs/querying/sql-functions.md

* applying suggestions

---------

Co-authored-by: Benedict Jin <asdf2014@apache.org>
2024-07-30 10:53:59 -07:00
Kashif Faraz 954aaafe0c
Refactor: Clean up compaction config classes (#16810)
Changes:
- Rename `CoordinatorCompactionConfig` to `DruidCompactionConfig`
- Rename `CompactionConfigUpdateRequest` to `ClusterCompactionConfig`
- Refactor methods in `DruidCompactionConfig`
- Clean up `DataSourceCompactionConfigHistory` and its tests
- Clean up tests and add new tests
- Change API path `/druid/coordinator/v1/config/global` to `/druid/coordinator/v1/config/cluster`
2024-07-30 12:17:25 +05:30
AmatyaAvadhanula 92a40d8169
Add API to fetch conflicting task locks (#16799)
* Add API to fetch conflicting active locks
2024-07-30 11:40:48 +05:30
Vishesh Garg e9ea243d97
Enable compaction ITs on MSQ engine (#16778)
Follow-up to #16291, this commit enables a subset of existing native compaction ITs on the MSQ engine.

In the process, the following changes have been introduced in the MSQ compaction flow:
- Populate `metricsSpec` in `CompactionState` from `querySpec` in `MSQControllerTask` instead of `dataSchema`
- Add check for pre-rolled-up segments having `AggregatorFactory` with different input and output column names
- Fix passing missing cluster-by clause in scan queries
- Add annotation of `CompactionState` to tombstone segments
2024-07-30 09:34:46 +05:30
Zoltan Haindrich c7cde31a89
HAVING clauses may not contain window functions (#16742)
Rejects having clauses if they contain windowed expressions.
Also added a check to produce a more descriptive error if an OVER expression
reaches the filter translation layer.

---------

Co-authored-by: Benedict Jin <asdf2014@apache.org>
2024-07-29 04:11:36 -04:00
dependabot[bot] f5527dc3e7
Bump io.grpc:grpc-netty-shaded from 1.57.2 to 1.65.1 (#16731)
Bumps [io.grpc:grpc-netty-shaded](https://github.com/grpc/grpc-java) from 1.57.2 to 1.65.1.
- [Release notes](https://github.com/grpc/grpc-java/releases)
- [Commits](https://github.com/grpc/grpc-java/compare/v1.57.2...v1.65.1)

---
updated-dependencies:
- dependency-name: io.grpc:grpc-netty-shaded
  dependency-type: direct:production
  update-type: version-update:semver-minor
...

Signed-off-by: dependabot[bot] <support@github.com>
Co-authored-by: dependabot[bot] <49699333+dependabot[bot]@users.noreply.github.com>
Co-authored-by: asdf2014 <asdf2014@apache.org>
2024-07-29 14:51:39 +08:00