Apache Kafka ingestion

When you enable the Kafka indexing service, you can configure supervisors on the Overlord to manage the creation and lifetime of Kafka indexing tasks.

Kafka indexing tasks read events using Kafka’s own partition and offset mechanism to guarantee exactly-once ingestion. The supervisor oversees the state of the indexing tasks to:

  • coordinate handoffs
  • manage failures
  • ensure that scalability and replication requirements are maintained.

This topic covers how to submit a supervisor spec to ingest event data, also known as message data, from Kafka. See the following for more information:

Kafka support

The Kafka indexing service supports transactional topics introduced in Kafka 0.11.x by default. The consumer for Kafka indexing service is incompatible with older Kafka brokers. If you are using an older version, refer to the Kafka upgrade guide.

Additionally, you can set isolation.level to read_uncommitted in consumerProperties if either:

  • You don’t need Druid to consume transactional topics.
  • You need Druid to consume older versions of Kafka. Make sure offsets are sequential, since there is no offset gap check in Druid anymore.

If your Kafka cluster enables consumer-group based ACLs, you can set group.id in consumerProperties to override the default auto generated group id.

Load the Kafka indexing service

To use the Kafka indexing service, load the druid-kafka-indexing-service extension on both the Overlord and the MiddleManagers. See Loading extensions for instructions on how to configure extensions.

Define a supervisor spec

Similar to the ingestion spec for batch ingestion, the supervisor spec configures the data ingestion for Kafka streaming ingestion. A supervisor spec has the following sections:

  • dataSchema to specify the Druid datasource name, primary timestamp, dimensions, metrics, transforms, and any necessary filters.
  • ioConfig to configure Kafka connection settings and configure how Druid parses the data. Kafka-specific connection details go in the consumerProperties. The ioConfig is also where you define the input format (inputFormat) of your Kafka data. For supported formats for Kafka and information on how to configure the input format, see Data formats.
  • tuningConfig to control various tuning parameters specific to each ingestion method. For a full description of all the fields and parameters in a Kafka supervisor spec, see the Kafka supervisor reference.

The following sections contain examples to help you get started with supervisor specs.

JSON input format supervisor spec example

The following example demonstrates a supervisor spec for Kafka that uses the JSON input format. In this case Druid parses the event contents in JSON format:

  1. {
  2. "type": "kafka",
  3. "spec": {
  4. "dataSchema": {
  5. "dataSource": "metrics-kafka",
  6. "timestampSpec": {
  7. "column": "timestamp",
  8. "format": "auto"
  9. },
  10. "dimensionsSpec": {
  11. "dimensions": [],
  12. "dimensionExclusions": [
  13. "timestamp",
  14. "value"
  15. ]
  16. },
  17. "metricsSpec": [
  18. {
  19. "name": "count",
  20. "type": "count"
  21. },
  22. {
  23. "name": "value_sum",
  24. "fieldName": "value",
  25. "type": "doubleSum"
  26. },
  27. {
  28. "name": "value_min",
  29. "fieldName": "value",
  30. "type": "doubleMin"
  31. },
  32. {
  33. "name": "value_max",
  34. "fieldName": "value",
  35. "type": "doubleMax"
  36. }
  37. ],
  38. "granularitySpec": {
  39. "type": "uniform",
  40. "segmentGranularity": "HOUR",
  41. "queryGranularity": "NONE"
  42. }
  43. },
  44. "ioConfig": {
  45. "topic": "metrics",
  46. "inputFormat": {
  47. "type": "json"
  48. },
  49. "consumerProperties": {
  50. "bootstrap.servers": "localhost:9092"
  51. },
  52. "taskCount": 1,
  53. "replicas": 1,
  54. "taskDuration": "PT1H"
  55. },
  56. "tuningConfig": {
  57. "type": "kafka",
  58. "maxRowsPerSegment": 5000000
  59. }
  60. }
  61. }

Kafka input format supervisor spec example

If you want to ingest data from other fields in addition to the Kafka message contents, you can use the kafka input format. The kafka input format lets you ingest:

  • the event key field
  • event headers
  • the Kafka event timestamp
  • the Kafka event value that stores the payload.

For example, consider the following structure for a message that represents a fictitious wiki edit in a development environment:

  • Event headers: {“environment”: “development”}
  • Event key: {“key: “wiki-edit”}
  • Event value: <JSON object with event payload containing the change details>
  • Event timestamp: “Nov. 10, 2021 at 14:06”

When you use the kafka input format, you configure the way that Druid names the dimensions created from the Kafka message:

  • headerLabelPrefix: Supply a prefix to the Kafka headers to avoid any conflicts with named dimensions. The default is kafka.header. Considering the header from the example, Druid maps the header to the following column: kafka.header.environment.
  • timestampColumnName: Supply a custom name for the Kafka timestamp in the Druid schema to avoid conflicts with other time columns. The default is kafka.timestamp.
  • keyColumnName: Supply the name for the Kafka key column in Druid. The default is kafka.key. Additionally, you must provide information about how Druid should parse the data in the Kafka message:
  • headerFormat: The default “string” decodes UTF8-encoded strings from the Kafka header. If you need another format, you can implement your own parser.
  • keyFormat: Takes a Druid inputFormat and uses the value for the first key it finds. According to the example the value is “wiki-edit”. It discards the key name in this case. If you store the key as a string, use the CSV input format. For example, if you have simple string for the the key wiki-edit, you can use the following to parse the key:

    1. "keyFormat": {
    2. "type": "csv",
    3. "hasHeaderRow": false,
    4. "findColumnsFromHeader": false,
    5. "columns": ["key"]
    6. }
  • valueFormat: Define how to parse the message contents. You can use any of the Druid input formats that work for Kafka.

For more information on data formats, see Data formats.

Finally, add the Kafka message columns to the dimensionsSpec. For the key and timestamp, you can use the dimension names you defined for keyColumnName and timestampColumnName. For header dimensions, append the header key to the headerLabelPrefix. For example kafka.header.environment.

The following supervisor spec demonstrates how to ingest the Kafka header, key, and timestamp into Druid dimensions:

  1. {
  2. "type": "kafka",
  3. "spec": {
  4. "ioConfig": {
  5. "type": "kafka",
  6. "consumerProperties": {
  7. "bootstrap.servers": "localhost:9092"
  8. },
  9. "topic": "wiki-edits",
  10. "inputFormat": {
  11. "type": "kafka",
  12. "headerLabelPrefix": "kafka.header.",
  13. "timestampColumnName": "kafka.timestamp",
  14. "keyColumnName": "kafka.key",
  15. "headerFormat": {
  16. "type": "string"
  17. },
  18. "keyFormat": {
  19. "type": "json"
  20. },
  21. "valueFormat": {
  22. "type": "json"
  23. },
  24. "findColumnsFromHeader": false
  25. },
  26. "useEarliestOffset": true
  27. },
  28. "tuningConfig": {
  29. "type": "kafka"
  30. },
  31. "dataSchema": {
  32. "dataSource": "wikiticker",
  33. "timestampSpec": {
  34. "column": "timestamp",
  35. "format": "posix"
  36. },
  37. "dimensionsSpec": {
  38. "dimensions": [
  39. {
  40. "type": "string",
  41. "name": "kafka.key"
  42. },
  43. {
  44. "type": "string",
  45. "name": "kafka.timestamp"
  46. },
  47. {
  48. "type": "string",
  49. "name": "kafka.header.environment"
  50. },
  51. "$schema",
  52. {
  53. "type": "long",
  54. "name": "id"
  55. },
  56. "type",
  57. {
  58. "type": "long",
  59. "name": "namespace"
  60. },
  61. "title",
  62. "comment",
  63. "user",]
  64. ]
  65. },
  66. "granularitySpec": {
  67. "queryGranularity": "none",
  68. "rollup": false,
  69. "segmentGranularity": "day"
  70. }
  71. }
  72. },
  73. "tuningConfig": {
  74. "type": "kafka"
  75. }
  76. }

After Druid ingests the data, you can query the Kafka message columns as follows:

  1. SELECT
  2. "kafka.header.environment",
  3. "kafka.key",
  4. "kafka.timestamp"
  5. FROM "wikiticker"
  6. kafka.header.environment kafka.key kafka.timestamp
  7. development wiki-edit 1636399229823

For more information, see kafka data format.

Submit a supervisor spec

Druid starts a supervisor for a dataSource when you submit a supervisor spec. You can use the data loader in the web console or you can submit a supervisor spec to the following endpoint:

http://<OVERLORD_IP>:<OVERLORD_PORT>/druid/indexer/v1/supervisor

For example:

  1. curl -X POST -H 'Content-Type: application/json' -d @supervisor-spec.json http://localhost:8090/druid/indexer/v1/supervisor

Where the file supervisor-spec.json contains your Kafka supervisor spec file.