mirror of https://github.com/apache/druid.git
Add nano-quickstart single server example configuration (#8390)
* Add nano-quickstart single server example configuration * Use two workers * Shrink processing buffers
This commit is contained in:
parent
02fe3db911
commit
c626452b47
|
@ -25,6 +25,7 @@ title: "Single server deployment"
|
|||
|
||||
Druid includes a set of reference configurations and launch scripts for single-machine deployments:
|
||||
|
||||
- `nano-quickstart`
|
||||
- `micro-quickstart`
|
||||
- `small`
|
||||
- `medium`
|
||||
|
@ -33,6 +34,8 @@ Druid includes a set of reference configurations and launch scripts for single-m
|
|||
|
||||
The `micro-quickstart` is sized for small machines like laptops and is intended for quick evaluation use-cases.
|
||||
|
||||
The `nano-quickstart` is an even smaller configuration, targeting a machine with 1 CPU and 4GB memory. It is meant for limited evaluations in resource constrained environments, such as small Docker containers.
|
||||
|
||||
The other configurations are intended for general use single-machine deployments. They are sized for hardware roughly based on Amazon's i3 series of EC2 instances.
|
||||
|
||||
The startup scripts for these example configurations run a single ZK instance along with the Druid services. You can choose to deploy ZK separately as well.
|
||||
|
@ -43,6 +46,11 @@ While example configurations are provided for very large single machines, at hig
|
|||
|
||||
## Single Server Reference Configurations
|
||||
|
||||
Nano-Quickstart: 1 CPU, 4GB RAM
|
||||
------------
|
||||
Launch command: `bin/start-nano-quickstart`
|
||||
Configuration directory: `conf/druid/single-server/nano-quickstart`
|
||||
|
||||
Micro-Quickstart: 4 CPU, 16GB RAM
|
||||
------------
|
||||
Launch command: `bin/start-micro-quickstart`
|
||||
|
|
|
@ -0,0 +1,24 @@
|
|||
#!/bin/bash -eu
|
||||
|
||||
# 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.
|
||||
|
||||
PWD="$(pwd)"
|
||||
WHEREAMI="$(dirname "$0")"
|
||||
WHEREAMI="$(cd "$WHEREAMI" && pwd)"
|
||||
|
||||
exec "$WHEREAMI/supervise" -c "$WHEREAMI/../conf/supervise/single-server/nano-quickstart.conf"
|
|
@ -0,0 +1,145 @@
|
|||
#
|
||||
# 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.
|
||||
#
|
||||
|
||||
# Extensions specified in the load list will be loaded by Druid
|
||||
# We are using local fs for deep storage - not recommended for production - use S3, HDFS, or NFS instead
|
||||
# We are using local derby for the metadata store - not recommended for production - use MySQL or Postgres instead
|
||||
|
||||
# If you specify `druid.extensions.loadList=[]`, Druid won't load any extension from file system.
|
||||
# If you don't specify `druid.extensions.loadList`, Druid will load all the extensions under root extension directory.
|
||||
# More info: https://druid.apache.org/docs/latest/operations/including-extensions.html
|
||||
druid.extensions.loadList=["druid-hdfs-storage", "druid-kafka-indexing-service", "druid-datasketches"]
|
||||
|
||||
# If you have a different version of Hadoop, place your Hadoop client jar files in your hadoop-dependencies directory
|
||||
# and uncomment the line below to point to your directory.
|
||||
#druid.extensions.hadoopDependenciesDir=/my/dir/hadoop-dependencies
|
||||
|
||||
|
||||
#
|
||||
# Hostname
|
||||
#
|
||||
druid.host=localhost
|
||||
|
||||
#
|
||||
# Logging
|
||||
#
|
||||
|
||||
# Log all runtime properties on startup. Disable to avoid logging properties on startup:
|
||||
druid.startup.logging.logProperties=true
|
||||
|
||||
#
|
||||
# Zookeeper
|
||||
#
|
||||
|
||||
druid.zk.service.host=localhost
|
||||
druid.zk.paths.base=/druid
|
||||
|
||||
#
|
||||
# Metadata storage
|
||||
#
|
||||
|
||||
# For Derby server on your Druid Coordinator (only viable in a cluster with a single Coordinator, no fail-over):
|
||||
druid.metadata.storage.type=derby
|
||||
druid.metadata.storage.connector.connectURI=jdbc:derby://localhost:1527/var/druid/metadata.db;create=true
|
||||
druid.metadata.storage.connector.host=localhost
|
||||
druid.metadata.storage.connector.port=1527
|
||||
|
||||
# For MySQL (make sure to include the MySQL JDBC driver on the classpath):
|
||||
#druid.metadata.storage.type=mysql
|
||||
#druid.metadata.storage.connector.connectURI=jdbc:mysql://db.example.com:3306/druid
|
||||
#druid.metadata.storage.connector.user=...
|
||||
#druid.metadata.storage.connector.password=...
|
||||
|
||||
# For PostgreSQL:
|
||||
#druid.metadata.storage.type=postgresql
|
||||
#druid.metadata.storage.connector.connectURI=jdbc:postgresql://db.example.com:5432/druid
|
||||
#druid.metadata.storage.connector.user=...
|
||||
#druid.metadata.storage.connector.password=...
|
||||
|
||||
#
|
||||
# Deep storage
|
||||
#
|
||||
|
||||
# For local disk (only viable in a cluster if this is a network mount):
|
||||
druid.storage.type=local
|
||||
druid.storage.storageDirectory=var/druid/segments
|
||||
|
||||
# For HDFS:
|
||||
#druid.storage.type=hdfs
|
||||
#druid.storage.storageDirectory=/druid/segments
|
||||
|
||||
# For S3:
|
||||
#druid.storage.type=s3
|
||||
#druid.storage.bucket=your-bucket
|
||||
#druid.storage.baseKey=druid/segments
|
||||
#druid.s3.accessKey=...
|
||||
#druid.s3.secretKey=...
|
||||
|
||||
#
|
||||
# Indexing service logs
|
||||
#
|
||||
|
||||
# For local disk (only viable in a cluster if this is a network mount):
|
||||
druid.indexer.logs.type=file
|
||||
druid.indexer.logs.directory=var/druid/indexing-logs
|
||||
|
||||
# For HDFS:
|
||||
#druid.indexer.logs.type=hdfs
|
||||
#druid.indexer.logs.directory=/druid/indexing-logs
|
||||
|
||||
# For S3:
|
||||
#druid.indexer.logs.type=s3
|
||||
#druid.indexer.logs.s3Bucket=your-bucket
|
||||
#druid.indexer.logs.s3Prefix=druid/indexing-logs
|
||||
|
||||
#
|
||||
# Service discovery
|
||||
#
|
||||
|
||||
druid.selectors.indexing.serviceName=druid/overlord
|
||||
druid.selectors.coordinator.serviceName=druid/coordinator
|
||||
|
||||
#
|
||||
# Monitoring
|
||||
#
|
||||
|
||||
druid.monitoring.monitors=["org.apache.druid.java.util.metrics.JvmMonitor"]
|
||||
druid.emitter=noop
|
||||
druid.emitter.logging.logLevel=info
|
||||
|
||||
# Storage type of double columns
|
||||
# ommiting this will lead to index double as float at the storage layer
|
||||
|
||||
druid.indexing.doubleStorage=double
|
||||
|
||||
#
|
||||
# Security
|
||||
#
|
||||
druid.server.hiddenProperties=["druid.s3.accessKey","druid.s3.secretKey","druid.metadata.storage.connector.password"]
|
||||
|
||||
|
||||
#
|
||||
# SQL
|
||||
#
|
||||
druid.sql.enable=true
|
||||
|
||||
#
|
||||
# Lookups
|
||||
#
|
||||
druid.lookup.enableLookupSyncOnStartup=false
|
|
@ -0,0 +1,33 @@
|
|||
<?xml version="1.0" encoding="UTF-8" ?>
|
||||
<!--
|
||||
~ 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.
|
||||
-->
|
||||
|
||||
<Configuration status="WARN">
|
||||
<Appenders>
|
||||
<Console name="Console" target="SYSTEM_OUT">
|
||||
<PatternLayout pattern="%d{ISO8601} %p [%t] %c - %m%n"/>
|
||||
</Console>
|
||||
</Appenders>
|
||||
<Loggers>
|
||||
<Root level="info">
|
||||
<AppenderRef ref="Console"/>
|
||||
</Root>
|
||||
</Loggers>
|
||||
</Configuration>
|
||||
|
|
@ -0,0 +1,10 @@
|
|||
-server
|
||||
-Xms512m
|
||||
-Xmx512m
|
||||
-XX:MaxDirectMemorySize=400m
|
||||
-XX:+ExitOnOutOfMemoryError
|
||||
-XX:+UseG1GC
|
||||
-Duser.timezone=UTC
|
||||
-Dfile.encoding=UTF-8
|
||||
-Djava.io.tmpdir=var/tmp
|
||||
-Djava.util.logging.manager=org.apache.logging.log4j.jul.LogManager
|
|
@ -0,0 +1 @@
|
|||
org.apache.druid.cli.Main server broker
|
|
@ -0,0 +1,38 @@
|
|||
#
|
||||
# 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.
|
||||
#
|
||||
|
||||
druid.service=druid/broker
|
||||
druid.plaintextPort=8082
|
||||
|
||||
# HTTP server settings
|
||||
druid.server.http.numThreads=6
|
||||
|
||||
# HTTP client settings
|
||||
druid.broker.http.numConnections=5
|
||||
druid.broker.http.maxQueuedBytes=5000000
|
||||
|
||||
# Processing threads and buffers
|
||||
druid.processing.buffer.sizeBytes=50000000
|
||||
druid.processing.numMergeBuffers=2
|
||||
druid.processing.numThreads=1
|
||||
druid.processing.tmpDir=var/druid/processing
|
||||
|
||||
# Query cache disabled -- push down caching and merging instead
|
||||
druid.broker.cache.useCache=false
|
||||
druid.broker.cache.populateCache=false
|
|
@ -0,0 +1,10 @@
|
|||
-server
|
||||
-Xms256m
|
||||
-Xmx256m
|
||||
-XX:+ExitOnOutOfMemoryError
|
||||
-XX:+UseG1GC
|
||||
-Duser.timezone=UTC
|
||||
-Dfile.encoding=UTF-8
|
||||
-Djava.io.tmpdir=var/tmp
|
||||
-Djava.util.logging.manager=org.apache.logging.log4j.jul.LogManager
|
||||
-Dderby.stream.error.file=var/druid/derby.log
|
|
@ -0,0 +1 @@
|
|||
org.apache.druid.cli.Main server coordinator
|
|
@ -0,0 +1,33 @@
|
|||
#
|
||||
# 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.
|
||||
#
|
||||
|
||||
druid.service=druid/coordinator
|
||||
druid.plaintextPort=8081
|
||||
|
||||
druid.coordinator.startDelay=PT10S
|
||||
druid.coordinator.period=PT5S
|
||||
|
||||
# Run the overlord service in the coordinator process
|
||||
druid.coordinator.asOverlord.enabled=true
|
||||
druid.coordinator.asOverlord.overlordService=druid/overlord
|
||||
|
||||
druid.indexer.queue.startDelay=PT5S
|
||||
|
||||
druid.indexer.runner.type=remote
|
||||
druid.indexer.storage.type=metadata
|
|
@ -0,0 +1,10 @@
|
|||
-server
|
||||
-Xms512m
|
||||
-Xmx512m
|
||||
-XX:MaxDirectMemorySize=400m
|
||||
-XX:+ExitOnOutOfMemoryError
|
||||
-XX:+UseG1GC
|
||||
-Duser.timezone=UTC
|
||||
-Dfile.encoding=UTF-8
|
||||
-Djava.io.tmpdir=var/tmp
|
||||
-Djava.util.logging.manager=org.apache.logging.log4j.jul.LogManager
|
|
@ -0,0 +1 @@
|
|||
org.apache.druid.cli.Main server historical
|
|
@ -0,0 +1,40 @@
|
|||
#
|
||||
# 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.
|
||||
#
|
||||
|
||||
druid.service=druid/historical
|
||||
druid.plaintextPort=8083
|
||||
|
||||
# HTTP server threads
|
||||
druid.server.http.numThreads=6
|
||||
|
||||
# Processing threads and buffers
|
||||
druid.processing.buffer.sizeBytes=50000000
|
||||
druid.processing.numMergeBuffers=2
|
||||
druid.processing.numThreads=1
|
||||
druid.processing.tmpDir=var/druid/processing
|
||||
|
||||
# Segment storage
|
||||
druid.segmentCache.locations=[{"path":"var/druid/segment-cache","maxSize":300000000000}]
|
||||
druid.server.maxSize=300000000000
|
||||
|
||||
# Query cache
|
||||
druid.historical.cache.useCache=true
|
||||
druid.historical.cache.populateCache=true
|
||||
druid.cache.type=caffeine
|
||||
druid.cache.sizeInBytes=50000000
|
|
@ -0,0 +1,9 @@
|
|||
-server
|
||||
-Xms64m
|
||||
-Xmx64m
|
||||
-XX:+ExitOnOutOfMemoryError
|
||||
-XX:+UseG1GC
|
||||
-Duser.timezone=UTC
|
||||
-Dfile.encoding=UTF-8
|
||||
-Djava.io.tmpdir=var/tmp
|
||||
-Djava.util.logging.manager=org.apache.logging.log4j.jul.LogManager
|
|
@ -0,0 +1 @@
|
|||
org.apache.druid.cli.Main server middleManager
|
|
@ -0,0 +1,39 @@
|
|||
#
|
||||
# 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.
|
||||
#
|
||||
|
||||
druid.service=druid/middleManager
|
||||
druid.plaintextPort=8091
|
||||
|
||||
# Number of tasks per middleManager
|
||||
druid.worker.capacity=2
|
||||
|
||||
# Task launch parameters
|
||||
druid.indexer.runner.javaOpts=-server -Xms256m -Xmx256m -XX:MaxDirectMemorySize=300m -Duser.timezone=UTC -Dfile.encoding=UTF-8 -XX:+ExitOnOutOfMemoryError -Djava.util.logging.manager=org.apache.logging.log4j.jul.LogManager
|
||||
druid.indexer.task.baseTaskDir=var/druid/task
|
||||
|
||||
# HTTP server threads
|
||||
druid.server.http.numThreads=6
|
||||
|
||||
# Processing threads and buffers on Peons
|
||||
druid.indexer.fork.property.druid.processing.numMergeBuffers=2
|
||||
druid.indexer.fork.property.druid.processing.buffer.sizeBytes=25000000
|
||||
druid.indexer.fork.property.druid.processing.numThreads=1
|
||||
|
||||
# Hadoop indexing
|
||||
druid.indexer.task.hadoopWorkingPath=var/druid/hadoop-tmp
|
|
@ -0,0 +1,10 @@
|
|||
-server
|
||||
-Xms128m
|
||||
-Xmx128m
|
||||
-XX:+UseG1GC
|
||||
-XX:MaxDirectMemorySize=128m
|
||||
-XX:+ExitOnOutOfMemoryError
|
||||
-Duser.timezone=UTC
|
||||
-Dfile.encoding=UTF-8
|
||||
-Djava.io.tmpdir=var/tmp
|
||||
-Djava.util.logging.manager=org.apache.logging.log4j.jul.LogManager
|
|
@ -0,0 +1 @@
|
|||
org.apache.druid.cli.Main server router
|
|
@ -0,0 +1,34 @@
|
|||
#
|
||||
# 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.
|
||||
#
|
||||
|
||||
druid.service=druid/router
|
||||
druid.plaintextPort=8888
|
||||
|
||||
# HTTP proxy
|
||||
druid.router.http.numConnections=25
|
||||
druid.router.http.readTimeout=PT5M
|
||||
druid.router.http.numMaxThreads=50
|
||||
druid.server.http.numThreads=50
|
||||
|
||||
# Service discovery
|
||||
druid.router.defaultBrokerServiceName=druid/broker
|
||||
druid.router.coordinatorServiceName=druid/coordinator
|
||||
|
||||
# Management proxy to coordinator / overlord: required for unified web console.
|
||||
druid.router.managementProxy.enabled=true
|
|
@ -0,0 +1,13 @@
|
|||
:verify bin/verify-java
|
||||
:verify bin/verify-default-ports
|
||||
:kill-timeout 10
|
||||
|
||||
!p10 zk bin/run-zk conf
|
||||
coordinator-overlord bin/run-druid coordinator-overlord conf/druid/single-server/nano-quickstart
|
||||
broker bin/run-druid broker conf/druid/single-server/nano-quickstart
|
||||
router bin/run-druid router conf/druid/single-server/nano-quickstart
|
||||
historical bin/run-druid historical conf/druid/single-server/nano-quickstart
|
||||
!p90 middleManager bin/run-druid middleManager conf/druid/single-server/nano-quickstart
|
||||
|
||||
# Uncomment to use Tranquility Server
|
||||
#!p95 tranquility-server tranquility/bin/tranquility server -configFile conf/tranquility/wikipedia-server.json -Ddruid.extensions.loadList=[]
|
Loading…
Reference in New Issue