diff --git a/.gitignore b/.gitignore index 18021663bc940..2451185f3ee6b 100644 --- a/.gitignore +++ b/.gitignore @@ -6,3 +6,4 @@ test_*/ *.local go.sum *.pyc +*.ezdraw diff --git a/README.md b/README.md index 124cb5d4c0c22..886e67c198911 100644 --- a/README.md +++ b/README.md @@ -1,66 +1,9 @@ # TiDB Lightning -TiDB Lightning is a data import tool which is used to fast import a large amount of data to the TiDB cluster. Currently, it only supports source data in the Mydumper file format and in the future it will support more formats like CSV. +**TiDB Lightning** is a tool for fast full import of large amounts of data into a TiDB cluster. +Currently, we support reading SQL dump exported via mydumper. -Now TiDB Lightning only supports full import of new tables. During the importing process, the cluster cannot provide services normally; as a result, TiDB Lightning is not suitable for importing data online. +![](docs/en_US/tidb-lightning.svg) -## TiDB Lightning architecture - -The following diagram shows the architecture of TiDB Lightning: - -![](media/tidb-lightning-architecture.png) - -One set of TiDB Lightning has two components: - -- `tidb-lightning` - - The front-end part of TiDB Lightning. It transforms the source data into Key-Value (KV) pairs and writes the data into `tikv-importer`. - -- `tikv-importer` - - The back-end part of TiDB Lightning. It caches, sorts, and splits the KV pairs written by `tidb-lightning` and imports the KV pairs to the TiKV cluster. - -## TiDB Lightning workflow - -1. Before importing data, `tidb-lightning` automatically switches the TiKV mode to the import mode via API. -2. `tidb-lightning` obtains data from the data source, transforms the source data into KV data, and then writes the data into `tikv-importer`. -3. When the data written by `tidb-lightning` reaches a specific size, `tidb-lightning` sends the `Import` command to `tikv-importer`. -4. `tikv-importer` divides and schedules the TiKV data of the target cluster and then imports the data to the TiKV cluster. -5. `tidb-lightning` transforms and imports the source data continuously until it finishes importing the data in the source data directory. -6. `tidb-lightning` performs the `Compact`, `Checksum`, and `Analyze` operation on tables in the target cluster. -7. `tidb-lightning` automatically switches the TiKV mode to the normal mode. Then the TiDB cluster can provide services normally. - -## Deploy process - -### Notes - -Before deploying TiDB Lightning, you should take note that: - -- When TiDB Lightning is running, the TiDB cluster cannot provide services normally. -- When you import data using TiDB Lightning, you cannot check some source data constraints such as the primary key conflict and unique index conflict. If needed, you can check using `ADMIN CHECK TABLE` via the MySQL client after importing, but it may take a long time. -- Currently, TiDB Lightning does not support breakpoint. If any error occurs during importing, delete the data from the target cluster using `DROP TABLE` and import the data again. -- If TiDB Lightning exits abnormally, you need to use the `-swtich-mode` command line parameter of `tidb-lightning` to manually close the import mode of the TiKV cluster and change it to the normal mode: - - ``` - ./bin/tidb-lightning -switch-mode normal - ``` - -### Hardware requirements - -See [Hardware requirements of TiDB Lightning](docs/tidb-lightning-user-guide.md#hardware-requirements) - -### Prepare - -Before importing, you should: - -- Deploy a set of TiDB cluster (TiDB version is 2.0.4 or later) which is the target cluster for importing (the target cluster). -- Prepare the binary file and the configuration file of `tikv-importer`. It is recommended to use standalone deployment. -- Prepare the binary file and the configuration file of `tidb-lightning`. It is recommended to use standalone deployment. - -Download the installation packages of `tikv-importer` and `tidb-lightning` via: - -https://download.pingcap.org/tidb-lightning-latest-linux-amd64.tar.gz - -### Deploy - -See [TiDB Lightning User Guide](docs/tidb-lightning-user-guide.md#deploy) +* [Detailed documentation](docs/en_US/README.md) +* [简体中文文档](docs/zh_CN/README.md) diff --git a/docs/en_US/01-Architecture.md b/docs/en_US/01-Architecture.md new file mode 100644 index 0000000000000..0e3aea402cf60 --- /dev/null +++ b/docs/en_US/01-Architecture.md @@ -0,0 +1,37 @@ +Architecture +============ + +![Architecture of TiDB Lightning tool set](./tidb-lightning.svg) + +The TiDB Lightning tool set consists of two components: + +- **`tidb-lightning`** (the "front end") reads the SQL dump and import the database structure + into the TiDB cluster, and also transforms the data into Key-Value (KV) pairs + and sends them to `tikv-importer`. + +- **`tikv-importer`** (the "back end") combines and sorts the KV pairs and then + imports these sorted pairs as a whole into the TiKV cluster. + +The complete import process is like this: + +1. Before importing, `tidb-lightning` switches the TiKV cluster to "import mode", which optimizes + the cluster for writing and disables automatic compaction. + +2. `tidb-lightning` creates the skeleton of all tables from the data source. + +3. For each table, `tidb-lightning` informs `tikv-importer` via gRPC to create an *engine file* + to store KV pairs. `tidb-lightning` then reads the SQL dump in parallel, transforms the data + into KV pairs according to the TiDB rules, and send them to `tikv-importer`'s engine files. + +4. Once a full table of KV pairs are received, `tikv-importer` divides and schedules these data + and imports them into the target TiKV cluster. + +5. `tidb-lightning` then performs a checksum comparison between the local data source and + those calculated from the cluster, to ensure there is no data corruption in the process. + +6. After all tables are imported, `tidb-lightning` performs a global compaction on the TiKV + cluster, and tell TiDB to `ANALYZE` all imported tables, to prepare for optimal query planning. + +7. Finally, `tidb-lightning` switches the TiKV cluster back to "normal mode" so the cluster + resumes normal services. + diff --git a/docs/en_US/02-Deployment.md b/docs/en_US/02-Deployment.md new file mode 100644 index 0000000000000..f8e9565c2b40f --- /dev/null +++ b/docs/en_US/02-Deployment.md @@ -0,0 +1,339 @@ +Deployment and Execution +======================== + +Hardware requirements +--------------------- + +### Notes + +Before starting TiDB Lightning, note that: + +- During the import process, the cluster cannot provide normal services. +- If `tidb-lightning` crashes, the cluster will be left in "import mode". + Forgetting to switch back to "normal mode" will lead to a high amount of uncompacted data on + the TiKV cluster, and will cause abnormally high CPU usage and stall. + You can manually switch the cluster back to "normal mode" via the `tidb-lightning-ctl` tool: + + ```sh + bin/tidb-lightning-ctl -switch-mode=normal + ``` + +### Deploying to separate machines + +`tidb-lightning` and `tikv-importer` are resource-intensive programs. It is recommended to deploy them into +two dedicated machines. + +To achieve the best performance, it is recommended to use the following hardware configuration: + +- `tidb-lightning` + + - 32+ logical cores CPU + - 16 GB+ memory + - 1 TB+ SSD, preferring higher read speed + - 10 Gigabit network card + - `tidb-lightning` fully consumes all CPU cores when running, + and deploying on a dedicated machine is highly recommended. + If not possible, `tidb-lightning` could be deployed together with other components like + `tidb-server`, and limiting the CPU usage via the `region-concurrency` setting. + +- `tikv-importer` + + - 32+ logical cores CPU + - 32 GB+ memory + - 1 TB+ SSD, preferring higher IOPS + - 10 Gigabit network card + - `tikv-importer` fully consumes all CPU, disk I/O and network bandwidth when running, + and deploying on a dedicated machine is strongly recommended. + If not possible, `tikv-importer` could be deployed together with other components like + `tikv-server`, but the import speed might be affected. + +If you have got enough machines, you could deploy multiple Lightning/Importer servers, +with each working on a distinct set of tables, to import the data in parallel. + +### Deploying to single machine + +If the hardware resources is severely under constraint, it is possible to deploy `tidb-lightning` +and `tikv-importer` and other components on the same machine, but note that the import performance +would also be impacted. + +We recommend the following configuration of the single machine: + +- 32+ logical cores CPU +- 32 GB+ memory +- 1 TB+ SSD, preferring higher IOPS +- 10 Gigabit network card + +`tidb-lightning` is a CPU intensive program. In an environment with mixed components, the resources +allocated to `tidb-lightning` must be limited. Otherwise, other components might not be able to run. +We recommend setting the `region-concurrency` to 75% of CPU logical cores. For instance, if the CPU +has 32 logical cores, the `region-concurrency` can be set to 24. + +Ansible deployment +------------------ + +TiDB Lightning can be deployed using Ansible, like [TiDB cluster itself][tidb-ansible]. + +[tidb-ansible]: https://github.com/pingcap/docs/blob/master/op-guide/ansible-deployment.md + +1. Edit `inventory.ini` to provide the addresses of the `tidb-lightning` and `tikv-importer` + servers: + + ```ini + ... + + [importer_server] + 192.168.20.9 + + [lightning_server] + 192.168.20.10 + + ... + ``` + +2. Configure these tools by editing the settings under `group_vars/*.yml`. + + * `group_vars/all.yml` + + ```yaml + ... + # The listening port of tikv-importer. Should be open to the tidb-lightning server. + tikv_importer_port: 20170 + ... + ``` + + * `group_vars/lightning_server.yml` + + ```yaml + --- + dummy: + + # The listening port for metrics gathering. Should be open to the monitoring servers. + tidb_lightning_pprof_port: 10089 + + # The file path tidb-lightning reads the mydumper SQL dump from. + data_source_dir: "{{ deploy_dir }}/mydumper" + ``` + + * `group_vars/importer_server.yml` + + ```yaml + --- + dummy: + + # The file path to store engine files. Should reside on a partition with large capacity. + import_dir: "{{ deploy_dir }}/data.import" + ``` + +3. Deploy the cluster via the usual steps + + ```sh + ansible-playbook bootstrap.yml + ansible-playbook deploy.yml + ``` + +4. Mount the data source to the path specified in the `data_source_dir` setting. + +5. Login to the `tikv-importer` server, and manually run + + ```sh + scripts/start_importer.sh + ``` + + to start Importer. + +6. Login to the `tidb-lightning` server, and manually run the following command + to start Lightning and import the data into the TiDB cluster. + + ```sh + scripts/start_lightning.sh + ``` + +7. After completion, run `scripts/stop_importer.sh` on the `tikv-importer` server to stop Importer. + +Manual deployment +----------------- + +### TiDB cluster + +Before importing, you should have deployed the TiDB cluster, with cluster version 2.0.4 or above. +Using the latest version is highly recommended. + +You can find deployment instructions in the +[TiDB Quick Start Guide](https://pingcap.com/docs/QUICKSTART/). + +Download the TiDB Lightning tool set (choose the one same as the cluster version): + +- **v2.1**: https://download.pingcap.org/tidb-lightning-release-2.1-linux-amd64.tar.gz +- **v2.0**: https://download.pingcap.org/tidb-lightning-release-2.0-linux-amd64.tar.gz + +### Starting `tikv-importer` + +1. Upload `bin/tikv-importer` from the tool set. + +2. Configure `tikv-importer.toml`: + + ```toml + # TiKV Importer configuration file template + + # Log file + log-file = "tikv-importer.log" + # Log level: trace, debug, info, warn, error, off. + log-level = "info" + + [server] + # Listening address of tikv-importer. tidb-lightning needs to connect to + # this address to write data. + addr = "0.0.0.0:20170" + # Size of thread pool for the gRPC server. + grpc-concurrency = 16 + + [metric] + # The Prometheus client push job name. + job = "tikv-importer" + # The Prometheus client push interval. + interval = "15s" + # The Prometheus Pushgateway address. + address = "" + + [rocksdb] + # The maximum number of concurrent background jobs. + max-background-jobs = 32 + + [rocksdb.defaultcf] + # Amount of data to build up in memory before flushing data to the disk. + write-buffer-size = "1GB" + # The maximum number of write buffers that are built up in memory. + max-write-buffer-number = 8 + + # The compression algorithms used in different levels. + # The algorithm at level-0 is used to compress KV data. + # The algorithm at level-6 is used to compress SST files. + # The algorithms at level-1 to level-5 are unused for now. + compression-per-level = ["lz4", "no", "no", "no", "no", "no", "zstd"] + + [import] + # The directory to store engine files. + import-dir = "/tmp/tikv/import" + # Number of threads to handle RPC requests. + num-threads = 16 + # Number of concurrent import jobs. + num-import-jobs = 24 + # Maximum duration to prepare regions. + #max-prepare-duration = "5m" + # Split regions into this size according to the importing data. + #region-split-size = "96MB" + # Stream channel window size, stream will be blocked on channel full. + #stream-channel-window = 128 + # Maximum number of open engines. + max-open-engines = 8 + ``` + +3. Run `tikv-importer`. + + ```sh + nohup ./tikv-importer -C tikv-importer.toml > nohup.out & + ``` + +### Starting `tidb-lightning` + +1. Upload `bin/tidb-lightning` and `bin/tidb-lightning-ctl` from the tool set. + +2. Mount the mydumper SQL dump onto the same machine. + +3. Configure `tidb-lightning.toml`: + + ```toml + ### tidb-lightning configuartion + + [lightning] + # HTTP port for debugging and Prometheus metrics pulling (0 to disable) + pprof-port = 10089 + + # check if the cluster satisfies the minimum requirement before starting + #check-requirements = true + + # The maximum number of tables to be handled concurrently. + # Must not exceed the max-open-engines setting for tikv-importer. + table-concurrency = 8 + # The concurrency number of data. It is set to the number of logical CPU + # cores by default. When deploying together with other components, you can + # set it to 75% of the size of logical CPU cores to limit the CPU usage. + #region-concurrency = + + # Logging + level = "info" + file = "tidb-lightning.log" + max-size = 128 # MB + max-days = 28 + max-backups = 14 + + [checkpoint] + # Whether to enable checkpoints. + # While importing, Lightning records which tables have been imported, so + # even if Lightning or other component crashed, we could start from a known + # good state instead of redoing everything. + enable = true + # The schema name (database name) to store the checkpoints + schema = "tidb_lightning_checkpoint" + # The data source name (DSN) in the form "USER:PASS@tcp(HOST:PORT)/". + # If not specified, the TiDB server from the [tidb] section will be used to + # store the checkpoints. You could also specify a different MySQL-compatible + # database server to reduce the load of the target TiDB cluster. + #dsn = "root@tcp(127.0.0.1:4000)/" + # Whether to keep the checkpoints after all data are imported. If false, the + # checkpoints will be deleted. Keeping the checkpoints can aid debugging but + # will leak metadata about the data source. + #keep-after-success = false + + [tikv-importer] + # The listening address of tikv-importer. Change it to the actual address + addr = "172.16.31.10:20170" + + [mydumper] + # Block size for file reading. Should be longer than the longest string of + # the data source. + read-block-size = 4096 # Byte (default = 4 KB) + # Each data file will be split into multiple chunks of this size. Each chunk + # will be processed in parallel. + region-min-size = 268435456 # Byte (default = 256 MB) + # mydumper local source data directory + data-source-dir = "/data/my_database" + # if no-schema is set true, tidb-lightning will assume the table skeletons + # already exists on the target TiDB cluster, and will not execute the CREATE + # TABLE statements + no-schema = false + + [tidb] + # Configuration of any one TiDB server from the cluster + host = "172.16.31.1" + port = 4000 + user = "root" + password = "" + # Table schema information is fetched from TiDB via this status-port. + status-port = 10080 + # Address of any one PD server from the cluster + pd-addr = "172.16.31.4:2379" + # tidb-lightning imports TiDB as a library and generates some logs itself. + # This setting controls the log level of the TiDB library. + log-level = "error" + # Sets TiDB session variable to speed up the Checksum and Analyze operations. + distsql-scan-concurrency = 16 + + # When data importing is complete, tidb-lightning can automatically perform + # the Checksum, Compact and Analyze operations. It is recommended to leave + # these as true in the production environment. + # The execution order: Checksum -> Compact -> Analyze + [post-restore] + # Performs `ADMIN CHECKSUM TABLE ` for each table to verify data integrity. + checksum = true + # Performs compaction on the TiKV cluster. + compact = true + # Performs `ANALYZE TABLE
` for each table. + analyze = true + ``` + +4. Run `tidb-lightning`. + + ```sh + nohup ./tidb-lightning -config tidb-lightning.toml > nohup.out & + ``` diff --git a/docs/en_US/03-Checkpoints.md b/docs/en_US/03-Checkpoints.md new file mode 100644 index 0000000000000..625eeb21aa65a --- /dev/null +++ b/docs/en_US/03-Checkpoints.md @@ -0,0 +1,105 @@ +Using Checkpoints +================= + +Importing a large database usually takes hours and days, and if such long running processes +spuriously crashes, it would be very time-wasting to redo the previously completed tasks. Lightning +uses *checkpoints* to stores the import progress, so that restarting `tidb-lightning` will continue +importing from where it lefts off. + +```toml +[checkpoint] +# Whether to enable checkpoints. +# While importing, Lightning will record which tables have been imported, so +# even if Lightning or other component crashed, we could start from a known +# good state instead of redoing everything. +enable = true + +# The schema name (database name) to store the checkpoints +schema = "tidb_lightning_checkpoint" + +# The data source name (DSN) in the form "USER:PASS@tcp(HOST:PORT)/". +# If not specified, the TiDB server from the [tidb] section will be used to +# store the checkpoints. You could also specify a different MySQL-compatible +# database server to reduce the load of the target TiDB cluster. +#dsn = "root@tcp(127.0.0.1:4000)/" + +# Whether to keep the checkpoints after all data are imported. If false, the +# checkpoints will be deleted. Keeping the checkpoints can aid debugging but +# will leak metadata about the data source. +#keep-after-success = false +``` + +Storage +------- + +Checkpoints are saved in any databases compatible with MySQL 5.7 or above, including MariaDB and +TiDB. By default the checkpoints are saved in the target database. + +While using the target database as the checkpoint storage, Lightning is importing large amount +of data at the same time. This puts extra stress on the target database and sometimes leads +to communication timeout. Therefore, **we strongly recommend you install a temporary MySQL server to +store these checkpoints**. This server can be installed on the same host as `tidb-lightning` and can +be uninstalled after the importer progress is completed. + +Checkpoint control +------------------ + +If `tidb-lightning` exits abnormally due to unrecoverable errors (e.g. data corruption), it will +refuse to reuse the checkpoints until the errors are resolved. This is to prevent worsening the +situation. The checkpoint errors can be resolved using the `tidb-lightning-ctl` program. + +### `--checkpoint-error-destroy` + +```sh +tidb-lightning-ctl --checkpoint-error-destroy='`schema`.`table` +``` + +If importing the table `` `schema`.`table` `` failed previously, this + +1. DROPs the table `` `schema`.`table` `` from the target database, i.e. removing all imported data. +2. resets the checkpoint record of this table to be "not yet started". + +If there is no errors involving the table `` `schema`.`table` ``, this operation does nothing. + +This option allows us to restarting importing the table from scratch. The schema and table names +must be quoted by backquotes and is case-sensitive. + +```sh +tidb-lightning-ctl --checkpoint-error-destroy=all +``` + +Same as applying the above on every table. This is the most convenient, safe and conservative +solution to fix the checkpoint error problem. + +### `--checkpoint-error-ignore` + +```sh +tidb-lightning-ctl --checkpoint-error-ignore='`schema`.`table`' +tidb-lightning-ctl --checkpoint-error-ignore=all +``` + +If importing the table `` `schema`.`table` `` failed previously, this clears the error status as +if nothing happened. The `all` variant applies this operation to all tables. + +This should only be used when you are sure that the error can indeed be ignored. If not, some +imported data could be lost. The only safety net is the final "checksum" check, and thus the +"checksum" option should always be enabled when using `--checkpoint-error-ignore`. + +### `--checkpoint-remove` + +```sh +tidb-lightning-ctl --checkpoint-remove='`schema`.`table`' +tidb-lightning-ctl --checkpoint-remove=all +``` + +Simply remove all checkpoint information about one table / all tables, regardless of their status. + +### `--checkpoint-dump` + +```sh +tidb-lightning-ctl --checkpoint-dump=output/directory +``` + +Dumps the content of the checkpoint into the given directory. Mainly used for debugging by technical +staff. + diff --git a/docs/en_US/04-Metrics.md b/docs/en_US/04-Metrics.md new file mode 100644 index 0000000000000..42b853addb50b --- /dev/null +++ b/docs/en_US/04-Metrics.md @@ -0,0 +1,124 @@ +Metrics +======= + +Both `tidb-lightning` and `tikv-importer` supports metrics collection via +[Prometheus](https://prometheus.io/). + +Configuration +------------- + +If you installed Lightning via TiDB-Ansible, simply adding the servers to the `[monitored_servers]` +section in the `inventory.ini` should be sufficient to let the Prometheus server to collect their +metrics. + +If you installed Lightning manually, follow the instructions below. + +### `tikv-importer` + +`tikv-importer` v2.1 uses [Pushgateway](https://github.com/prometheus/pushgateway) to deliver +metrics. Configure `tikv-importer.toml` to recognize the Pushgateway with the following settings: + +```toml +[metric] + +# The Prometheus client push job name. +job = "tikv-importer" + +# The Prometheus client push interval. +interval = "15s" + +# The Prometheus Pushgateway address. +address = "" +``` + +### `tidb-lightning` + +The metrics of `tidb-lightning` can be gathered directly by Prometheus as long as it is discovered. +The metrics port can be set in `tidb-lightning.toml` + +```toml +[lightning] +# HTTP port for debugging and Prometheus metrics pulling (0 to disable) +pprof-port = 10089 + +... +``` + +Prometheus needs to be configured to discover the `tidb-lightning` server. For instance, you could +hard-code the server address to the `scrape_configs` section: + +```yaml +... +scrape_configs: + - job_name: 'tidb-lightning' + static_configs: + - targets: ['192.168.20.10:10089'] +``` + +Raw metrics +----------- + +### `tikv-importer` + +Metrics provided by `tikv-importer` are listed under the namespace `tikv_import_*`. + +* **`tikv_import_rpc_duration`** (Histogram) + + Bucketed histogram of import RPC duration. Labels: + + * **request**: RPC name, e.g. `open_engine`, `import_engine`, etc. + * **result**: `ok` / `error` + +* **`tikv_import_write_chunk_bytes`** (Histogram) + + Bucketed histogram of import write chunk bytes. + +* **`tikv_import_write_chunk_duration`** (Histogram) + + Bucketed histogram of import write chunk duration. + +* **`tikv_import_upload_chunk_bytes`** (Histogram) + + Bucketed histogram of import upload chunk bytes. + +* **`tikv_import_upload_chunk_duration`** (Histogram) + + Bucketed histogram of import upload chunk duration. + +### `tidb-lightning` + +Metrics provided by `tidb-lightning` are listed under the namespace `lightning_*`. + +* **`lightning_importer_engine`** (Counter) + + Counting open and closed engine files. Labels: + + * **type**: `open` / `closed` + +* **`lightning_idle_workers`** (Gauge) + + Counting idle workers. Values should be less than the `table-concurrency`/`region-concurrency` + settings and are typically be zero. Labels: + + * **name**: `table` / `region` + +* **`lightning_kv_encoder`** (Counter) + + Counting open and closed KV encoders. KV encoders are in-memory TiDB instances which converts + SQL INSERT statements into KV pairs. The net values should be bounded in a healthy situation. + Labels: + + * **type**: `open` / `closed` + +* **`lightning_tables`** (Counter) + + Counting number of tables processed and their status. Labels: + + * **state**: `pending` / `written` / `closed` / `imported` / `altered_auto_inc` / `checksum` / `completed` + * **result**: `success` / `failure` + +* **`lightning_chunks`** (Counter) + + Counting number of chunks processed and their status. Labels: + + * **state**: `estimated` / `pending` / `running` / `finished` / `failed` diff --git a/docs/en_US/05-Errors.md b/docs/en_US/05-Errors.md new file mode 100644 index 0000000000000..b7746e9c0599b --- /dev/null +++ b/docs/en_US/05-Errors.md @@ -0,0 +1,69 @@ +Common Errors +============= + +When Lightning encounters an unrecoverable error, it exits with nonzero exit code and leaves the +reason in the log file. Errors are typically printed at the end of the log. You can also search for +the string `[error]` to look for non-fatal errors. + +Here are some commonly encountered errors in the `tidb-lightning` log file and their resolution. + +## checksum failed: checksum mismatched remote vs local + +**Cause**: The checksum of a table in the local data source and the remote imported database differ. +There are several deeper reasons of this error: + +1. The table might have already been populated before. These old data would affect the final checksum. + +2. If the table does not have an integer PRIMARY KEY, some rows might be imported repeatedly between + checkpoints. This is a known bug to be fixed in the next release. + +3. If the remote checksum is 0 i.e. nothing is imported, it is possible that the cluster is too hot + and failed to take it any data. + +4. If the data is mechanically generated, ensure it respects the constrains of the table: + + * AUTO_INCREMENT columns should be positive, and should not contain the value "0". + * There should be no duplicated entries in UNIQUE and PRIMARY KEYs. + +**Solutions**: + +1. Delete the corrupted data with `tidb-lightning-ctl --error-checkpoint-destroy=all`, and restart + Lightning to import the affected tables again. + +2. Consider using an external database to store the checkpoints (change `[checkpoint] dsn`) to + reduce the target database's load. + +## ResourceTemporarilyUnavailable("Too many open engines …: 8") + +**Cause**: The number of concurrent engine files exceeds the limit imposed by `tikv-importer`. This +could be caused by misconfiguration. Additionally, if `tidb-lightning` exited abnormally, an engine +file might be left at a dangling open state, which could cause this error as well. + +**Solutions**: + +1. Increase the value of `max-open-engine` setting in `tikv-importer.toml`. This value is typically + dictated by the available memory. This could be calculated as: + + > Max Memory Usage ≈ `max-open-engine` × `write-buffer-size` × `max-write-buffer-number` + +2. Decrease the value of `table-concurrency` so it is less than `max-open-engine`. + +3. Restart `tikv-importer` to forcefully remove all engine files. This also removes all + partially imported tables, thus running `tidb-lightning-ctl --error-checkpoint-destroy=all` is + required. + +## cannot guess encoding for input file, please convert to UTF-8 manually + +**Cause**: Lightning only recognizes the UTF-8 and GB-18030 encodings for the table schemas. This +error is emitted if the file isn't in any of these encodings. It is also possible that the file has +mixed encoding e.g. contains a string in UTF-8 and another string in GB-18030, due to historical +`ALTER TABLE` executions. + +**Solutions**: + +1. Fix the schema so that the file is entirely in either UTF-8 or GB-18030. + +2. Manually CREATE the affected tables in the target database, then set + `[mydumper] no-schema = true` to skip automatic table creation. + + diff --git a/docs/en_US/06-FAQ.md b/docs/en_US/06-FAQ.md new file mode 100644 index 0000000000000..6dd93e206320e --- /dev/null +++ b/docs/en_US/06-FAQ.md @@ -0,0 +1,127 @@ +Frequently Asked Questions +========================== + +What is the minimum TiDB/TiKV/PD cluster version supported by Lightning? +------------------------------------------------------------------------ + +The minimal version is 2.0.4. + +Does Lightning support importing multiple schemas (databases)? +-------------------------------------------------------------- + +Yes. + +What is the privilege requirements for the target database? +----------------------------------------------------------- + +Lightning requires the following privileges: + +* SELECT +* UPDATE +* ALTER +* CREATE +* DROP + +If the target database is used to store checkpoints, it additionally requires these privileges: + +* INSERT +* DELETE + +Lightning encountered an error when importing one table. Will it affect other tables? Will the process be terminated? +--------------------------------------------------------------------------------------------------------------------- + +If only one table has an error encountered, the rest will still be processed normally. + +How to ensure integrity of the imported data? +--------------------------------------------- + +Lightning by default performs checksum on the local data source and the imported tables. If +there is checksum mismatch, the process would be aborted. These checksum information can be read +from the log. + +You could also execute the `ADMIN CHECKSUM TABLE` SQL command on the target table to recompute the +checksum of the imported data. + +```text +mysql> ADMIN CHECKSUM TABLE `schema`.`table`; ++---------+------------+---------------------+-----------+-------------+ +| Db_name | Table_name | Checksum_crc64_xor | Total_kvs | Total_bytes | ++---------+------------+---------------------+-----------+-------------+ +| schema | table | 5505282386844578743 | 3 | 96 | ++---------+------------+---------------------+-----------+-------------+ +1 row in set (0.01 sec) +``` + +What kind of data source format is supported by Lightning? +---------------------------------------------------------- + +In version 2.1.0 we only support SQL dump generated by +[mydumper](https://github.com/pingcap/mydumper) stored in the local filesystem. + +Could Lightning skip creating schema and tables? +------------------------------------------------ + +Yes. If you have already created the tables in the target database, you could set `no-schema = true` +in the `[data-source]` section in `tidb-lightning.toml`. This makes Lightning skips the +`CREATE TABLE` invocations and fetch the metadata directly from the target database. Lightning will +exit with error if a table is actually missing. + +Can the Strict SQL Mode be disabled to allow importing invalid data? +-------------------------------------------------------------------- + +Yes. By default [`sql_mode`] used by Lightning is `"STRICT_TRANS_TABLES,NO_ENGINE_SUBSTITUTION"`, +which disallows invalid data such as the date `1970-00-00`. The mode can be changed by modifying the +`sql-mode` setting in the `[tidb]` section in `tidb-lightning.toml`. + +```toml +... +[tidb] +sql-mode = "" +... +``` + +[`sql_mode`]: https://dev.mysql.com/doc/refman/5.7/en/sql-mode.html + +Can one `tikv-importer` serve multiple `tidb-lightning` instances? +------------------------------------------------------------------ + +Yes, as long as every `tidb-lightning` instance operate on different tables. + +How to stop `tikv-importer`? +---------------------------- + +If it is deployed using TiDB-Ansible, run `scripts/stop_importer.sh` under the deployed folder. + +Otherwise, obtain the process ID with `ps aux | grep tikv-importer`, then run `kill «pid»`. + +How to stop `tidb-lightning`? +----------------------------- + +If it is deployed using TiDB-Ansible, run `scripts/stop_lightning.sh` under the deployed folder. + +If `tidb-lightning` is running in foreground, simply press Ctrl+C to stop it. + +Otherwise, obtain the process ID with `ps aux | grep tidb-importer`, then run `kill -2 «pid»`. + +Why `tidb-lightning` suddenly quits while running in background? +---------------------------------------------------------------- + +It is potentially caused by starting `tidb-lightning` incorrectly, causing the system to send a +SIGHUP signal to stop it. If this is the case, there should be a log entry like: + +``` +2018/08/10 07:29:08.310 main.go:47: [info] Got signal hangup to exit. +``` + +We do not recommend using `nohup` directly in the command line. Rather, put the `nohup` inside a +script file and execute the script. + +Why my TiDB cluster is using lots of CPU and running very slowly after using Lightning? +--------------------------------------------------------------------------------------- + +If `tidb-lightning` abnormally exited, the cluster might be stuck in the "import mode", which is not +suitable for production. You can force the cluster back to "normal mode" with: + +```sh +tidb-lightning-ctl --switch-mode=normal +``` diff --git a/docs/en_US/README.md b/docs/en_US/README.md new file mode 100644 index 0000000000000..aaf62a29269ac --- /dev/null +++ b/docs/en_US/README.md @@ -0,0 +1,26 @@ +TiDB Lightning +============== + +**TiDB Lightning** is a tool for fast full import of large amounts of data into a TiDB cluster. +Currently, we support reading SQL dump exported via mydumper. + +1. [Architecture](01-Architecture.md) +2. [Deployment and Execution](02-Deployment.md) +3. [Using Checkpoints](03-Checkpoints.md) +4. [Metrics](04-Metrics.md) +5. [Common Errors](05-Errors.md) +6. [FAQ](06-FAQ.md) + +## Notes + +Before starting TiDB Lightning, note that: + +- During the import process, the cluster cannot provide normal services. +- If `tidb-lightning` crashes, the cluster will be left in "import mode". + Forgetting to switch back to "normal mode" will lead to a high amount of uncompacted data on + the TiKV cluster, and will cause abnormally high CPU usage and stall. + You can manually switch the cluster back to "normal mode" via the `tidb-lightning-ctl` tool: + + ```sh + bin/tidb-lightning-ctl -switch-mode=normal + ``` diff --git a/docs/en_US/tidb-lightning.svg b/docs/en_US/tidb-lightning.svg new file mode 100644 index 0000000000000..d84e7f8834612 --- /dev/null +++ b/docs/en_US/tidb-lightning.svg @@ -0,0 +1,551 @@ + + + + + + + + + + + + + + + + + + + image/svg+xml + + + + + + + + + + + + + data + source + + + tidb-lightning + + tikv-importer + + tikv-server + + tidb-server + + tidb-server + + pd-server + + pd-server + + pd-server + + tikv-server + + tikv-server + + + + + + KV + pairs + + + + SQL + dump + + TiDB Lightning Tool Set + TiDB Cluster + TiKV Cluster + PD Cluster + + + + Sorted + + KV + pairs + + + + + + DDL, Analysis + + + + + meta- + data + + + + + diff --git a/docs/tidb-lightning-user-guide.md b/docs/tidb-lightning-user-guide.md deleted file mode 100644 index 5cf2da8c36258..0000000000000 --- a/docs/tidb-lightning-user-guide.md +++ /dev/null @@ -1,238 +0,0 @@ -# TiDB Lightning User Guide - -TiDB Lightning is a data import tool which is used to fast import a large amount of data to the TiDB cluster. Currently, it only supports source data in the Mydumper file format and in the future it will support more formats like CSV. - -Now TiDB Lightning only supports full import of new tables. During the importing process, the cluster cannot provide services normally; as a result, TiDB Lightning is not suitable for importing data online. - -## TiDB Lightning architecture - -The following diagram shows the architecture of TiDB Lightning: - -![](./media/tidb-lightning-architecture.png) - -One set of TiDB Lightning has two components: - -- `tidb-lightning` - - The front-end part of TiDB Lightning. It transforms the source data into Key-Value (KV) pairs and writes the data into `tikv-importer`. - -- `tikv-importer` - - The back-end part of TiDB Lightning. It caches, sorts, and divides the KV pairs written by `tidb-lightning` and imports the KV pairs to the TiKV cluster. - -## TiDB Lightning workflow - -1. Before importing data, `tidb-lightning` automatically switches the TiKV mode to the import mode via API. -2. `tidb-lightning` obtains data from the data source, transforms the source data into KV data, and then writes the data into `tikv-importer`. -3. When the data written by `tidb-lightning` reaches a specific size, `tidb-lightning` sends the `Import` command to `tikv-importer`. -4. `tikv-importer` divides and schedules the TiKV data of the target cluster and then imports the data to the TiKV cluster. -5. `tidb-lightning` transforms and imports the source data continuously until it finishes importing the data in the source data directory. -6. `tidb-lightning` performs the `Compact`, `Checksum`, and `Analyze` operation on tables in the target cluster. -7. `tidb-lightning` automatically switches the TiKV mode to the normal mode. Then the TiDB cluster can provide services normally. - -## Deploy process - -### Notes - -Before deploying TiDB Lightning, you should take note that: - -- When TiDB Lightning is running, the TiDB cluster cannot provide services normally. -- When you import data using TiDB Lightning, you cannot check some source data constraints such as the primary key conflict and unique index conflict. If needed, you can check using `ADMIN CHECK TABLE` via the MySQL client after importing, but it may take a long time. -- Currently, TiDB Lightning does not support breakpoint. If any error occurs during importing, delete the data from the target cluster using `DROP TABLE` and import the data again. -- If TiDB Lightning exits abnormally, you need to use the `-swtich-mode` command line parameter of `tidb-lightning` to manually close the import mode of the TiKV cluster and change it to the normal mode: - - ``` - ./bin/tidb-lightning -switch-mode normal - ``` - -### Hardware requirements - -#### Hardware requirements for separate deployment - -The following are the hardware requirements for deploying one set of TiDB Lighting. If you have enough machines, you can deploy multiple sets of TiDB Lightning, divide the source code based on the table grain and then import the data concurrently. - -- tidb-lightning - - - 32+ logical core CPU - - 16 GB+ memory - - 1 TB+ SSD - - 10 Gigabit network card - - Need to be deployed separately from the online business because TiDB Lighting fully consumes the CPU during runtime. Under certain circumstances, you can deploy it along with another component (like `tidb-server`) on one machine and configure to limit the CPU usage of `tidb-lightning`. See the `region-concurrency` part in the first step of [Deploy `tidb-lightning`](#deploy-tidb-lightning). - -- tikv-importer - - - 32+ logical core CPU - - 32 GB+ memory - - 1 TB+ SSD - - 10 Gigabit network card - - Need to be deployed separately from the online business because TiDB Lighting fully consumes the CPU, I/O and the network bandwidth during runtime. Under certain circumstances, you can deploy it along with other component (like `tikv-server`) on one machine, but the importing speed may be affected. - -#### Hardware requirements for mixed deployment - -Under certain circumstances, you can deploy `tidb-lightning` and `tikv-importer` (or another application) mixedly on one machine. - -- 32+ logical core CPU -- 32 GB+ memory -- 1 TB+ SSD -- 10 Gigabit network card - -> **Note:** `tidb-lightning` is CPU intensive. If you use mixed deployment for it, you need to configure `region-concurrency` to limit the number of occupied CPU cores of `tidb-lightning`. Otherwise other applications may be affected. - -> You can configure the `region-concurrency` parameter of `tidb-lightning` to allocate 75% of CPU resources to `tidb-lightning`. For example, if CPU has 32 logical cores, you can set `region-concurrency` to 24. - -### Prepare - -Before importing, you should: - -- Deploy a set of TiDB cluster (TiDB version is 2.0.4 or later) which is the target cluster for importing (the target cluster). -- Prepare the binary file and the configuration file of `tikv-importer`. It is recommended to use standalone deployment. -- Prepare the binary file and the configuration file of `tidb-lightning`. It is recommended to use standalone deployment. - -Download the installation packages of `tikv-importer` and `tidb-lightning` via: - -https://download.pingcap.org/tidb-lightning-latest-linux-amd64.tar.gz - -### Deploy - -#### Deploy the TiDB cluster - -For details, see [Deploy TiDB Using Ansible](https://pingcap.com/docs/op-guide/ansible-deployment/). - -#### Deploy `tikv-importer` - -1. Configure `tikv-importer.toml. - - ``` - # TiKV Importer configuration file template - - # log file. - log-file = "tikv-importer.log" - # log level: trace, debug, info, warn, error, off. - log-level = "info" - - [server] - # the listening address of tikv-importer. tidb-lightning needs to connect to this address to write data. Set it to the actual IP address. - addr = "172.16.30.4:20170" - # size of thread pool for the gRPC server. - grpc-concurrency = 16 - - [metric] - # the Prometheus client push job name. - job = "tikv-importer" - # the Prometheus client push interval. - interval = "15s" - # the Prometheus Pushgateway address. - address = "" - - [rocksdb] - # the maximum number of concurrent background jobs. - max-background-jobs = 32 - - [rocksdb.defaultcf] - # amount of data to build up in memory before flushing data to the disk. - write-buffer-size = "1GB" - # the maximum number of write buffers that are built up in memory. - max-write-buffer-number = 8 - - # the compression algorithms used in different levels. - # the algorithm at level-0 is used to compress KV data. - # the algorithm at level-6 is used to compress SST files. - # the algorithms at level-1 ~ level-5 are not used now. - compression-per-level = ["lz4", "no", "no", "no", "no", "no", "zstd"] - - [import] - # this directory is used to store the data written by `tidb-lightning`. - import-dir = "/tmp/tikv/import" - # the number of threads to handle RPC requests. - num-threads = 16 - # the number of concurrent import jobs. - num-import-jobs = 24 - # the stream channel window size. Stream will be blocked when the channel is full. - stream-channel-window = 128 - ``` - -2. Run the executable file of `tikv-importer`. - - ``` - nohup ./tikv-importer -C tikv-importer.toml > nohup.out & - ``` - -#### Deploy `tidb-lightning` - -1. Configure `tidb-lightning.toml`. - - ``` - ### tidb-lightning configuration - [lightning] - - # background profile for debugging ( 0 to disable ) - pprof-port = 10089 - - - # table-concurrency controls the maximum handled tables concurrently while reading Mydumper SQL files. It can affect the tikv-importer memory usage amount. - table-concurrency = 8 - # region-concurrency changes the concurrency number of data. It is set to the number of logical CPU cores by default and needs no configuration. - # in mixed configuration, you can set it to 75% of the size of logical CPU cores. - # region-concurrency default to runtime.NumCPU() - # region-concurrency = - - - # logging - level = "info" - file = "tidb-lightning.log" - max-size = 128 # MB - max-days = 28 - max-backups = 14 - - [tikv-importer] - # the listening address of tikv-importer. Change it to the actual address in tikv-importer.toml. - addr = "172.16.31.4:20170" - # size of batch to import KV data into TiKV: xxx (GB) - batch-size = 500 # GB - - [mydumper] - # block size of file reading - read-block-size = 4096 # Byte (default = 4 KB) - # divide source data file into multiple Region/chunk to execute restoring in parallel - region-min-size = 268435456 # Byte (default = 256 MB) - # the source data directory of Mydumper. tidb-lightning will automatically create the corresponding database and tables based on the schema file in the directory. - data-source-dir = "/data/mydumper" - # If no-schema is set to true, tidb-lightning will obtain the table schema information from tidb-server, instead of creating the database or tables based on the schema file of data-source-dir. This applies to manually creating tables or the situation where the table schema exits in TiDB. - no-schema = false - - - - # configuration for TiDB (pick one of them if it has many TiDB servers) and the PD server. - [tidb] - # the target cluster information - # the listening address of tidb-server. Setting one of them is enough. - host = "127.0.0.1" - port = 4000 - user = "root" - password = "" - # table schema information is fetched from TiDB via this status-port. - status-port = 10080 - # the listening address of pd-server. Setting one of them is enough. - pd-addr = "127.0.0.1:2379" - # Lightning uses some code of TiDB (used as a library) and the flag controls its log level. - log-level = "error" - # set TiDB session variable to speed up performing the Checksum or Analyze operation on the table. - distsql-scan-concurrency = 16 - - # when data importing is finished, tidb-lightning can automatically perform the Checksum, Compact and Analyze operations. - # it is recommended to set it to true in the production environment. - # the execution order: Checksum -> Compact -> Analyze - [post-restore] - if it is set to true, tidb-lightning will perform the ADMIN CHECKSUM TABLE
operation on the tables one by one. - checksum = true - # if it is set to true, tidb-lightning will perform a full Compact operation on all the data. If the Compact operation fails, you can use ./bin/tidb-lightning -compact or the command of tikv-ctl to compact the data manually. - compact = true - # if it is set to true, tidb-lightning will perform the ANALYZE TABLE
operation on the tables one by one. If the Analyze operation fails, you can analyze data manually on the Mysql client. - analyze = true - ``` - -2. Run the executable file of `tidb-lightning`. - - ``` - nohup ./tidb-lightning -config tidb-lightning.toml > nohup.out & - ``` diff --git a/docs/zh_CN/01-Architecture.md b/docs/zh_CN/01-Architecture.md new file mode 100644 index 0000000000000..ac031fd04d965 --- /dev/null +++ b/docs/zh_CN/01-Architecture.md @@ -0,0 +1,33 @@ +软件架构 +======= + +![TiDB Lightning 其整体架构](./tidb-lightning.svg) + +TiDB Lightning 主要包含两个部分: + +- **`tidb-lightning`** (“前端”) 主要完成适配工作,通过读取 SQL dump,在下游 TiDB 集群建表、 + 将数据转换成键/值对 (KV 对) 发送到 `tikv-importer`、检查数据完整性等等。 + +- **`tikv-importer`** (“后端”) 主要完成数据导入 TiKV 集群的工作,把 `tidb-lightning` 写入的 KV 对缓存、排序、 + 切分并导入到 TiKV 集群。 + +整体工作原理如下: + +1. 在导数据之前,`tidb-lightning` 会自动将 TiKV 集群切换为“导入模式” (import mode), + 优化写入效率并停止自动压缩 (compaction)。 + +2. `tidb-lightning` 会在目标数据库建立架构和表,并获取其元数据。 + +3. `tidb-lightning` 会通过 gRPC 让 `tikv-importer` 为对每一张表准备一个“引擎文件 (engine file)” + 来处理 KV 对。`tidb-lightning` 会并发读取 SQL dump,将数据源转换成与 TiDB 相同编码的 KV 对, + 然后发送到 `tikv-importer` 里对应的引擎文件。 + +4. 当一整个表完全导入到 `tikv-importer`,它便开始对目标 TiKV 集群数据进行分裂和调度,然后导入数据到 TiKV 集群。 + +5. 之后,`tidb-lightning` 会对比本地数据源及下游集群的校验和 (checksum),确保导入的数据无损。 + +6. 整个数据库完成导入后,`tidb-lightning` 便会让 TiKV 集群进行一次全局压缩,以及让 TiDB 分析 (`ANALYZE`) + 这些新增的数据,以优化日后的操作。 + +7. 在所有步骤完毕后,`tidb-lightning` 自动将 TiKV 切换回“普通模式” (normal mode), + 此后 TiDB 集群可以正常对外提供服务。 diff --git a/docs/zh_CN/02-Deployment.md b/docs/zh_CN/02-Deployment.md new file mode 100644 index 0000000000000..11cd7a819da82 --- /dev/null +++ b/docs/zh_CN/02-Deployment.md @@ -0,0 +1,320 @@ +部署与执行 +======== + +硬件需求 +------- + +### 注意事项 + +在使用 TiDB Lightning 前,请注意: + +- TiDB Lightning 运行后,TiDB 集群将无法正常对外提供服务。 +- 若 `tidb-lightning` 崩溃,集群会留在“导入模式”。若忘记转回“普通模式”,集群会产生大量未压缩的文件, + 继而消耗 CPU 并导致迟延 (stall)。此时需要使用 `tidb-lightning-ctl` 手动把集群转回“普通模式”: + + ```sh + bin/tidb-lightning-ctl -switch-mode=normal + ``` + +### 单独部署 + +`tidb-lightning` 和 `tikv-importer` 皆为资源密集程序,建议各自单独部署。 + +为了优化效能,建议硬件配置如下: + +- `tidb-lightning` + + - 32+ 逻辑核 CPU + - 16 GB+ 内存 + - 1 TB+ SSD 硬盘,读取速度越快越好。 + - 使用万兆网卡 + - 运行过程默认会打满 CPU,建议单独部署。条件不允许的情况下可以和其他组件 (比如 `tidb-server`) 部署在同一台机器, + 然后通过配置 `region-concurrency` 限制 `tidb-lightning` 的 CPU 使用。 + +- `tikv-importer` + + - 32+ 逻辑核 CPU + - 32 GB+ 内存 + - 1 TB+ SSD 硬盘,IOPS 越高越好。 + - 使用万兆网卡 + - 运行过程中 CPU、IO 和网络带宽都可能打满,建议单独部署。条件不允许的情况下可以和其他组件 (比如 `tikv-server`) + 部署在同一台机器,但是可能会影响导入速度。 + +如果机器充裕的话,可以部署多套 `tidb-lightning` + `tikv-importer`,然后把源数据根据表为粒度进行切分,并发导入。 + +### 混合部署 + +如果条件有限,可以将 `tidb-lightning` 和 `tikv-importer` (也可以是其他程序) 混合部署在同一台机器上, +但注意这样会影响导入速度。 + +使用混合部署建议的硬件配置如下: + +- 32+ 逻辑核 CPU +- 32 GB+ 内存 +- 1 TB+ SSD 硬盘,IOPS 越高越好。 +- 使用万兆网卡 + +`tidb-lightning` 是 CPU 密集型程序,如果和其他程序混合部署,需要通过 `region-concurrency` +限制下 `tidb-lightning` 的 实际占用 CPU 核数,否则会影响其他程序的正常运行。建议将混合部署机器 75% 的 CPU 分配给 +`tidb-lightning`。例如,机器为 32 核,则 `tidb-lightning` 的 `region-concurrency` 可设为 24。 + +Ansible 部署 +----------- + +TiDB Lightning 可随 TiDB 集群一起以 [Ansible 部署][tidb-ansible]。 + +[tidb-ansible]: https://github.com/pingcap/docs-cn/blob/master/op-guide/ansible-deployment.md + +1. 编辑 `inventory.ini`,分别配置一个 IP 来部署 `tidb-lightning` 和 `tikv-importer` + + ```ini + ... + + [importer_server] + 192.168.20.9 + + [lightning_server] + 192.168.20.10 + + ... + ``` + +2. 修改 `group_vars/*.yml` 的变量配置这两个工具。 + + * `group_vars/all.yml` + + ```yaml + ... + # tikv-importer 的监听端口。需对 tidb-lightning 服务器开放。 + tikv_importer_port: 20170 + ... + ``` + + * `group_vars/lightning_server.yml` + + ```yaml + --- + dummy: + + # 提供监控告警的端口。需对监控服务器 (monitoring_server) 开放。 + tidb_lightning_pprof_port: 10089 + + # 获取 mydumper SQL dump 的路径。 + data_source_dir: "{{ deploy_dir }}/mydumper" + ``` + + * `group_vars/importer_server.yml` + + ```yaml + --- + dummy: + + # 储存引擎文件的路径。需存放在空间足够大的分区。 + import_dir: "{{ deploy_dir }}/data.import" + ``` + +3. 开始部署。 + + ```sh + ansible-playbook bootstrap.yml + ansible-playbook deploy.yml + ``` + +4. 将数据源写入 `data_source_dir` 指定的路径。 + +5. 登录 `tikv-importer` 的服务器,并执行 + + ```sh + scripts/start_importer.sh + ``` + + 来启动 Importer。 + +6. 登录 `tidb-lightning` 的服务器,并执行 + + ```sh + scripts/start_lightning.sh + ``` + + 来启动 Lightning,开始导入过程。 + +7. 完成后,在 `tikv-importer` 的服务器执行 `scripts/stop_importer.sh` 来关闭 Importer。 + +手动部署 +------- + +### TiDB 集群 + +在开始导入之前,请先部署一套要进行导入的 TiDB 集群 (版本要求 2.0.4 以上)。建议使用最新版。 + +部署方法可参考 [TiDB 快速入门指南](https://pingcap.com/docs-cn/QUICKSTART/)。 + +TiDB Lightning 安装包可以通过下面的下载链接获取 (请选择与集群相同的版本): + +- **v2.1**: https://download.pingcap.org/tidb-lightning-release-2.1-linux-amd64.tar.gz +- **v2.0**: https://download.pingcap.org/tidb-lightning-release-2.0-linux-amd64.tar.gz + +### 启动 `tikv-importer` + +1. 从安装包上传 `bin/tikv-importer` + +2. 配置 `tikv-importer.toml` + + ```toml + # TiKV Importer 配置文件模版 + + # 日志文件。 + log-file = "tikv-importer.log" + # 日志等级:trace、debug、info、warn、error、off。 + log-level = "info" + + [server] + # tikv-importer 监听的地址,tidb-lightning 需要连到这个地址进行数据写入。 + addr = "0.0.0.0:20170" + # gRPC 服务器的线程池大小 + grpc-concurrency = 16 + + [metric] + # 给 Prometheus 客户端的推送任务名称。 + job = "tikv-importer" + # 给 Prometheus 客户端的推送间隔。 + interval = "15s" + # Prometheus Pushgateway 地址。 + address = "" + + [rocksdb] + # 最大的背景任务并发数。 + max-background-jobs = 32 + + [rocksdb.defaultcf] + # 数据在刷新到硬盘前能存于内存的容量上限。 + write-buffer-size = "1GB" + # 存于内存的写入缓冲最大数量。 + max-write-buffer-number = 8 + + # 各个压缩层级使用的算法。 + # 第 0 层的算法用于压缩 KV 数据。 + # 第 6 层的算法用于压缩 SST 文件。 + # 第 1 至 5 层的算法目前忽略。 + compression-per-level = ["lz4", "no", "no", "no", "no", "no", "zstd"] + + [import] + # 存储引擎文档 (engine file) 的文档夹路径。 + import-dir = "/tmp/tikv/import" + # 处理 gRPC 请求的线程数量。 + num-threads = 16 + # 导入任务并发数。 + num-import-jobs = 24 + # 预处理 Region 最长时间。 + #max-prepare-duration = "5m" + # 把要导入的数据切分为这个大小的 Region。 + #region-split-size = "96MB" + # 流管道窗口大小,管道满时会阻塞流。 + #stream-channel-window = 128 + # 引擎文档同时打开的最大数量。 + max-open-engines = 8 + ``` + +3. 运行 `tikv-importer` + + ```sh + nohup ./tikv-importer -C tikv-importer.toml > nohup.out & + ``` + +### 启动 `tidb-lightning` + +1. 从安装包上传 `bin/tidb-lightning` 及 `bin/tidb-lightning-ctl`。 + +2. 将 mydumper SQL dump 数据源写入到同样的机器。 + +3. 配置 `tidb-lightning.toml` + + ```toml + # TiDB Lightning 配置文件模版 + + [lightning] + # 用于除错和 Prometheus 监控的 HTTP 端口。输入 0 关闭。 + pprof-port = 10089 + + # 开始导入前先检查集群版本是否支持。 + #check-requirements = true + + # 控制同时处理的表的数量。这个值会影响 tikv-importer 的内存使用量。 + # 不能超過 tikv-importer 中 max-open-engines 的值。 + table-concurrency = 8 + # 转换数据的并发数,默认为逻辑 CPU 数量,不需要配置。 + # 混合部署的情况下可以配置为逻辑 CPU 的 75% 大小。 + #region-concurrency = + + # 日志 + level = "info" + file = "tidb-lightning.log" + max-size = 128 # MB + max-days = 28 + max-backups = 14 + + [checkpoint] + # 启用断点续传。 + # 导入时,Lightning 会记录当前进度。 + # 若 Lightning 或其他组件异常退出,在重启时可以避免重复再导入已完成的数据。 + enable = true + # 存储断点的数据库名称 + schema = "tidb_lightning_checkpoint" + # 存储断点的数据库连接参数 (DSN),格式为“用户:密码@tcp(地址:端口)”。 + # 默认会重用 [tidb] 设置目标数据库来存储断点。 + # 为避免加重目标集群的压力,建议另外使用一个兼容 MySQL 协议的数据库服务器。 + #dsn = "root@tcp(127.0.0.1:4000)/" + # 导入成功后是否保留断点。默认为删除。 + # 保留断点可用于除错,但有可能泄漏数据源的元数据。 + #keep-after-success = false + + [tikv-importer] + # tikv-importer 的监听地址。请改成 tikv-importer 服务器的实际地址 + addr = "172.16.31.10:20170" + + [mydumper] + # 文件读取区块大小。 + read-block-size = 4096 # 字节 (默认 = 4 KB) + # 每个文档在转换时会切分为多个 Chunk 并发处理,此为每个 Chunk 的大小。 + region-min-size = 268435456 # 字节 (默认 = 256 MB) + # mydumper 源数据目录。 + data-source-dir = "/data/my_database" + # 如果 no-schema 设置为 true,tidb-lightning 将直接去 tidb-server 获取表结构信息, + # 而不是根据 data-source-dir 的 schema 文件来创建库/表, + # 适用于手动创建建表或者 tidb 本来就有表结构的情况。 + no-schema = false + + [tidb] + # 目标集群的信息。tidb-server 的监听地址,填一个即可。 + host = "172.16.31.1" + port = 4000 + user = "root" + password = "" + # 表架构信息在从 TiDB 的“状态端口”获取。 + status-port = 10080 + # pd-server 的监听地址,填一个即可。 + pd-addr = "172.16.31.4:2379" + # tidb-lightning 引用了 TiDB 库,而它自己会产生一些日志。此设置控制 TiDB 库的日志等级。 + log-level = "error" + # 设置 TiDB 会话变量,提升 CHECKSUM 和 ANALYZE 的速度。 + distsql-scan-concurrency = 16 + + # 导完数据以后可以自动进行校验和 (CHECKSUM)、压缩 (Compact) 和分析 (ANALYZE) 的操作。 + # 生产环境建议都设为 true + # 执行顺序是: CHECKSUM -> Compact -> ANALYZE。 + [post-restore] + # 如果设置为 true, 会对每个表逐个做 `ADMIN CHECKSUM TABLE
` 操作 + checksum = true + # 如果设置为 true,会对所有数据做一次全量 Compact。 + compact = true + # 如果设置为 true,会对每个表逐个做 `ANALYZE TABLE
` 操作。 + analyze = true + ``` + +4. 运行 `tidb-lightning`。如果直接在命令行中用 nohup 启动程序,可能会因为 SIGHUP 信号而退出, + 建议把nohup 放到脚本里面,如 + + ```sh + #!/bin/bash + nohup ./tidb-lightning -config tidb-lightning.toml > nohup.out & + ``` diff --git a/docs/zh_CN/03-Checkpoints.md b/docs/zh_CN/03-Checkpoints.md new file mode 100644 index 0000000000000..85895176baeae --- /dev/null +++ b/docs/zh_CN/03-Checkpoints.md @@ -0,0 +1,93 @@ +断点续传 +======= + +导入大量的数据一般耗时数小时至数天,这么长时间运行的进程会有一定机率非正常中断。如果每次重启都从头开始, +就会浪费掉之前已成功导入的数据。Lightning 提供了“断点续传”的功能,即使 `tidb-lightning` 崩溃, +在重启时仍然从之前的进度继续工作。 + +```toml +[checkpoint] +# 启用断点续传。 +# 导入时,Lightning 会记录当前进度。 +# 若 Lightning 或其他组件异常退出,在重启时可以避免重复再导入已完成的数据。 +enable = true + +# 存储断点的架构名称 (数据库名称) +schema = "tidb_lightning_checkpoint" + +# 存储断点的数据库连接参数 (DSN),格式为“用户:密码@tcp(地址:端口)”。 +# 默认会重用 [tidb] 设置目标数据库来存储断点。 +# 为避免加重目标集群的压力,建议另外使用一个兼容 MySQL 的数据库服务器。 +#dsn = "root@tcp(127.0.0.1:4000)/" + +# 导入成功后是否保留断点。默认为删除。 +# 保留断点可用于除错,但有可能泄漏数据源的元数据。 +#keep-after-success = false +``` + +存储 +---- + +Lightning 的断点可以存放在任何兼容 MySQL 5.7 或以上的数据库,包括 MariaDB 和 TiDB。 +默认在没有选择下会存在目标数据库里。 + +目标数据库在导入期间会有大量的操作,若使用目标数据库来存储断点会加重其负担,甚至有可能造成通信超时丢失数据。因此, +我们强烈建议另外部署一台兼容 MySQL 的临时数据库服务器。此数据库也可以安装在 `tidb-lightning` 的主机上。 +导入完毕后可以删除。 + +控制 +---- + +若 `tidb-lightning` 因不可回复的错误而退出 (例如数据出错),重启时不会使用断点,而是直接报错离开。 +为保证已导入的数据安全,这些错误必须先解决掉才能继续。使用 `tidb-lightning-ctl` 工具可以标示已经收复。 + +### `--checkpoint-error-destroy` + +```sh +tidb-lightning-ctl --checkpoint-error-destroy='`schema`.`table` +``` + +如果导入 `` `schema`.`table` `` 这个表曾经出错,这条命令会 + +1. 从目标数据库移除 (DROP) 这个表,清除已导入的数据。 +2. 将断点重设到“未开始”的状态。 + +如果 `` `schema`.`table` `` 没有出错则无操作。 + +此命令让失败的表从头开始整个导入过程。选项中的架构和表名必须以反引号 (`` ` ``) 包裹,而且区分大小写。 + +```sh +tidb-lightning-ctl --checkpoint-error-destroy=all +``` + +传入“all”会对所有表进行上述操作。这是最方便、安全但保守的断点错误解决方法。 + +### `--checkpoint-error-ignore` + +```sh +tidb-lightning-ctl --checkpoint-error-ignore='`schema`.`table`' +tidb-lightning-ctl --checkpoint-error-ignore=all +``` + +如果导入 `` `schema`.`table` `` 这个表曾经出错,这条命令会清除出错状态如同没事发生过一样。 +传入“all”会对所有表进行上述操作。 + +除非确定错误是的确可以忽略,否则不要使用这个选项。如果错误是真实的话,可能会导致数据不完全。 +启用校验和 (CHECKSUM) 可以防止数据出错被忽略。 + +### `--checkpoint-remove` + +```sh +tidb-lightning-ctl --checkpoint-remove='`schema`.`table`' +tidb-lightning-ctl --checkpoint-remove=all +``` + +不论有否出错,把表的断点清除。 + +### `--checkpoint-dump` + +```sh +tidb-lightning-ctl --checkpoint-dump=output/directory +``` + +将所有断点备份到传入的文件夹。主要用作技术支持。 diff --git a/docs/zh_CN/04-Metrics.md b/docs/zh_CN/04-Metrics.md new file mode 100644 index 0000000000000..3b401641d8dec --- /dev/null +++ b/docs/zh_CN/04-Metrics.md @@ -0,0 +1,118 @@ +监控告警 +======= + +TiDB Lightning 支持给 [Prometheus](https://prometheus.io/) 采集度量 (metrics)。 + +配置 +---- + +如使用 TiDB Ansible 部署 Lightning, +只要将服务器地址加到 `inventory.ini` 里的 `[monitored_servers]` 部份即可。 + +若然手动部署,请参照以下步骤。 + +### `tikv-importer` + +`tikv-importer` v2.1 使用 [Pushgateway](https://github.com/prometheus/pushgateway) 来推送度量。 +需要配置 `tikv-importer.toml` 来连接 Pushgateway: + +```toml +[metric] + +# 给 Prometheus 客户端的推送任务名称。 +job = "tikv-importer" + +# 给 Prometheus 客户端的推送间隔。 +interval = "15s" + +# Prometheus Pushgateway 地址。 +address = "" +``` + +### `tidb-lightning` + +只要 Prometheus 能发现 `tidb-lightning` 的监控地址就能收集度量。监控的端口可从 `tidb-lightning.toml` 配置: + +```toml +[lightning] +# 用于除错和 Prometheus 监控的 HTTP 端口。输入 0 关闭。 +pprof-port = 10089 + +... +``` + +让 Prometheus 发现 Lightning 其中一个方法是把地址直接写入其配置文件,例如: + +```yaml +... +scrape_configs: + - job_name: 'tidb-lightning' + static_configs: + - targets: ['192.168.20.10:10089'] +``` + +度量 +---- + +### `tikv-importer` + +`tikv-importer` 的度量皆以 `tikv_import_*` 为前缀。 + +* **`tikv_import_rpc_duration`** (直方图) + + 导入 RPC 需时的直方图。标签: + + * **request**: RPC 名称,如 `open_engine`、`import_engine` 等 + * **result**: `ok` / `error` + +* **`tikv_import_write_chunk_bytes`** (直方图) + + 从 `tidb-lightning` 写入每个区块大小的直方图。 + +* **`tikv_import_write_chunk_duration`** (直方图) + + 从 `tidb-lightning` 写入每个区块需时直方图。 + +* **`tikv_import_upload_chunk_bytes`** (直方图) + + 上传到 TiKV 的每个区块大小的直方图。 + +* **`tikv_import_upload_chunk_duration`** (直方图) + + 上传到 TiKV 的每个区块需时的直方图。 + +### `tidb-lightning` + +`tidb-lightning` 的度量皆以 `tikv_import_*` 为前缀。 + +* **`lightning_importer_engine`** (计数器) + + 计算已开启及关闭的引擎文档数量。标签: + + * **type**: `open` / `closed` + +* **`lightning_idle_workers`** (测量仪) + + 计算闲置的工作流程。数值应低于 `table-concurrency`/`region-concurrency`,而经常为零。标签: + + * **name**: `table` / `region` + +* **`lightning_kv_encoder`** (计数器) + + 计算已开启及关闭的 KV 编码器。KV 编码器是运行于内存的 TiDB 实例,用于将 SQL 的 INSERT 语句转换成 KV 对。 + 此度量的净值 (开启减掉关闭) 在正常情况下不应持续增长。标签: + + * **type**: `open` / `closed` + +* **`lightning_tables`** (计数器) + + 计算处理过的表及其状态。标签: + + * **state**: `pending` / `written` / `closed` / `imported` / `altered_auto_inc` / `checksum` / `completed` + * **result**: `success` / `failure` + +* **`lightning_chunks`** (计数器) + + 计算处理过的 Chunks 及其状态。标签: + + * **state**: `estimated` / `pending` / `running` / `finished` / `failed` diff --git a/docs/zh_CN/05-Errors.md b/docs/zh_CN/05-Errors.md new file mode 100644 index 0000000000000..ee931ad151b52 --- /dev/null +++ b/docs/zh_CN/05-Errors.md @@ -0,0 +1,56 @@ +错误排解 +======= + +当 Lightning 遇到不可回复的错误时便会异常退出,并在日志记下错误原因。一般会在日志底部找到。 +亦可以搜索 `[error]` 字串找出中间发生的错误。 + +以下为一些常见的错误及其解决方法。 + +## checksum failed: checksum mismatched remote vs local + +**原因**: 本地数据源跟目标数据库某个表的校验和不一致。这通常有更深层的原因: + +1. 这张表可能本身已有数据,影响最终结果 + +2. 如果表没有整数型主键 (integer PRIMARY KEY),在断点续传后有些行可能会被重复导入。这是已知的问题,在下个版本会修正。 + +3. 如果目标数据库的校验和全是 0,代表甚么也没导进去,有可能是集群太忙无法接收导入的指令。 + +4. 如果数据源是由机器生成而不是从 mydumper 备份的,请确保数据符合表的限制,例如: + + * 自增 (AUTO_INCREMENT) 的列需要正数,不能为 0。 + * 单一键和主键 (UNIQUE and PRIMARY KEYs) 不能有重复的值。 + +**解决办法**: + +1. 使用 `tidb-lightning-ctl --error-checkpoint-destroy=all` 把出错的表删除, + 然后重启 Lightning 重新导入那些表。 + +2. 把断点存放在外部数据库 (修改 `[checkpoint] dsn`),减轻目标集群压力。 + +## ResourceTemporarilyUnavailable("Too many open engines …: 8") + +**原因**: 并行打开的引擎文件 (engine files) 超出 `tikv-importer` 里的限制。这可能由配置错误引起。即使配置没问题, +如果 `tidb-lightning` 曾经异常退出,亦有可能令引擎文件残留在打开的状态,占据可用的数量。 + +**解决办法**: + +1. 提高 `tikv-importer.toml` 内 `max-open-engine` 的值。这个设置主要由内存决定,计算公式为: + + > 最大内存使用量 ≈ `max-open-engine` × `write-buffer-size` × `max-write-buffer-number` + +2. 降低 `table-concurrency`,使之低于 `max-open-engine`。 + +3. 重启 `tikv-importer` 来强制移除所有引擎文件。这样亦会丢弃导入了一半的表,所以启动 Lightning 前必须使用 + `tidb-lightning-ctl --error-checkpoint-destroy=all`。 + +## cannot guess encoding for input file, please convert to UTF-8 manually + +**原因**: Lightning 只支持 UTF-8 和 GB-18030 编码的表架构。此错误代表数据源不是这里任一个编码。 +亦有可能是文件中混合了不同的编码,例如因为在不同的环境运行过 ALTER TABLE,令表架构同时出现 UTF-8 和 GB-18030 的字符。 + +**解决办法**: + +1. 编辑数据源,保存为纯 UTF-8 或 GB-18030 的文件。 + +2. 自行在目标数量库创建所有的表,然后设置 `[mydumper] no-schema = true` 跳过创表的步骤。 diff --git a/docs/zh_CN/06-FAQ.md b/docs/zh_CN/06-FAQ.md new file mode 100644 index 0000000000000..1d60851ef7185 --- /dev/null +++ b/docs/zh_CN/06-FAQ.md @@ -0,0 +1,119 @@ +常见问题 +======= + +Lightning 对 TiDB/TiKV/PD 的最低版本要求是多少? +-------------------------------------------- + +最低版本要求是 2.0.4。 + +Lightning 支持导入多个库吗? +------------------------- + +支持。 + +Lightning 对下游的数据库账号权限要求是? +----------------------------------- + +Lightning 需要以下权限: + +* SELECT +* UPDATE +* ALTER +* CREATE +* DROP + +另外,存储断点的数据库额外需要以下权限: + +* INSERT +* DELETE + +Lightning 在导数据过程中某个表报错了,会影响其他表吗?进程会马上退出吗? +------------------------------------------------------------ + +如果只是个别表报错,不会影响整体,报错的那个表会停止处理,继续处理其他的表。 + +如何校验导入的数据的正确性? +----------------------- + +Lightning 默认会对导入数据计算校验和 (checksum),如果校验和不一致就会停止导入该表。可以在日志看到相关的信息。 + +TiDB 亦支持从 mysql 命令行运行 `ADMIN CHECKSUM TABLE` 指令计算校验和。 + +```text +mysql> ADMIN CHECKSUM TABLE `schema`.`table`; ++---------+------------+---------------------+-----------+-------------+ +| Db_name | Table_name | Checksum_crc64_xor | Total_kvs | Total_bytes | ++---------+------------+---------------------+-----------+-------------+ +| schema | table | 5505282386844578743 | 3 | 96 | ++---------+------------+---------------------+-----------+-------------+ +1 row in set (0.01 sec) +``` + +Lightning 支持哪些格式的数据源? +---------------------------- + +到 v2.1.0 版本为止,只支持本地文档形式的数据源,支持 [mydumper](https://github.com/pingcap/mydumper) 格式。 + +我已经在下游创建好库和表了,Lightning 可以忽略建库建表操作吗? +---------------------------------------------------- + +可以。在配置文档中的 `[data-source]` 将 `no-schema` 设置为 `true` 即可。 +`no-schema=true` 会默认下游已经创建好所需的数据库和表,如果没有创建,会报错。 + +有些不合法的数据,能否通过关掉严格 SQL 模式 (Strict SQL MOde) 来导入? +------------------------------------------------------------- + +可以。Lightning 默认的 [`sql_mode`] 为 `"STRICT_TRANS_TABLES,NO_ENGINE_SUBSTITUTION"`。 +这个设置不允许一些非法的数值,例如 `1970-00-00` 这样的日期。可以修改配置文件 `[tidb]` 下的 `sql-mode` 值。 + +```toml +... +[tidb] +sql-mode = "" +... +``` + +[`sql_mode`]: https://dev.mysql.com/doc/refman/5.7/en/sql-mode.html + +可以起一个 `tikv-importer`,同时有多个 `tidb-lightning` 进程导入数据吗? +----------------------------------------------------------------- + +只要每个 Lightning 操作的表互不相同就可以。 + +如何正确关闭 `tikv-importer` 进程? +-------------------------------- + +如使用 TiDB Ansible 部署,在 Importer 的服务器上运行 `scripts/stop_importer.sh` 即可。 + +否则,可通过 `ps aux | grep tikv-importer` 获取进程ID,然后 `kill «pid»`。 + +如何正确关闭 `tidb-lightning` 进程? +--------------------------------- + +如使用 TiDB Ansible 部署,在 Lightning 的服务器上运行 `scripts/stop_lightning.sh` 即可。 + +如果 `tidb-lightning` 正在前台运行,可直接按 Ctrl+C 退出。 + +否则,可通过 `ps aux | grep tidb-lightning` 获取进程ID,然后 `kill -2 «pid»`。 + +进程在服务器上运行,进程莫名其妙地就退出了,是怎么回事呢? +----------------------------------------------- + +这种情况可能是启动方式不正确,导致因为收到 SIGHUP 信号而退出,此时 `tidb-lightning.log` 通常有这幺一行日志: + +``` +2018/08/10 07:29:08.310 main.go:47: [info] Got signal hangup to exit. +``` + +不推荐直接在命令行中使用 `nohup` 启动进程,而应该把 `nohup` 这行命令放到一个脚本中运行。 + +为什么用过 Lightning 之后,TiDB 集群变得又慢又耗 CPU? +------------------------------------------------ + +如果 `tidb-lightning` 曾经异常退出,集群可能仍留在“导入模式” (import mode),不适合在生产环境工作。 +此时需要强制切换回“普通模式” (normal mode): + +```sh +tidb-lightning-ctl --switch-mode=normal +``` + diff --git a/docs/zh_CN/README.md b/docs/zh_CN/README.md new file mode 100644 index 0000000000000..4a926e6661927 --- /dev/null +++ b/docs/zh_CN/README.md @@ -0,0 +1,23 @@ +TiDB Lightning +============== + +**TiDB Lightning** 是一个全量数据高速导入到 TiDB 集群的工具。目前支持 mydumper 输出格式的数据源。 + +1. [软件架构](01-Architecture.md) +2. [部署执行](02-Deployment.md) +3. [断点续传](03-Checkpoints.md) +4. [监控告警](04-Metrics.md) +5. [错误排解](05-Errors.md) +6. [常见问题](06-FAQ.md) + +## 注意事项 + +在使用 TiDB Lightning 前,请注意: + +- TiDB Lightning 运行后,TiDB 集群将无法正常对外提供服务。 +- 若 `tidb-lightning` 崩溃,集群会留在“导入模式”。若忘记转回“普通模式”,集群会产生大量未压缩的文件, + 继而消耗 CPU 并导致迟延 (stall)。此时需要使用 `tidb-lightning-ctl` 手动把集群转回“普通模式”: + + ```sh + bin/tidb-lightning-ctl -switch-mode=normal + ``` diff --git a/docs/zh_CN/tidb-lightning.svg b/docs/zh_CN/tidb-lightning.svg new file mode 100644 index 0000000000000..9d4acbe32a452 --- /dev/null +++ b/docs/zh_CN/tidb-lightning.svg @@ -0,0 +1,521 @@ + + + + + + + + + + + + + + + + + + + image/svg+xml + + + + + + + + + + + + + 数据源 + + + tidb-lightning + + tikv-importer + + tikv-server + + tidb-server + + tidb-server + + pd-server + + pd-server + + pd-server + + tikv-server + + tikv-server + + + + + KV对 + + + SQL + Dump + + TiDB Lightning 工具包 + TiDB 集群 + TiKV 集群 + PD 集群 + + + + 排好 + 序的 + KV对 + + + + + + + DDL、
 + 数据分析 + + + + + 元数据 + + + + diff --git a/media/tidb-lightning-architecture.png b/media/tidb-lightning-architecture.png deleted file mode 100644 index 82ff9af96da8c..0000000000000 Binary files a/media/tidb-lightning-architecture.png and /dev/null differ