vinothchandar commented on code in PR #5436: URL: https://github.com/apache/hudi/pull/5436#discussion_r881010366
########## rfc/rfc-51/rfc-51.md: ########## @@ -0,0 +1,234 @@ +<!-- + 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. +--> + +# RFC-50: Hudi CDC + +# Proposers + +- @Yann Byron + +# Approvers + +- @Raymond + +# Statue +JIRA: [https://issues.apache.org/jira/browse/HUDI-3478](https://issues.apache.org/jira/browse/HUDI-3478) + +# Hudi Supports Change-Data-Capture + +## Abstract + +We want to introduce the Change-Data-Capture(CDC) capacity that make hudi can produce the changeing data by which we can know how the records is changed, to response the CDC cases. + + +## Background + +In some use cases where hudi tables is used as streaing source, We want to be aware every records' changing in one commit exactly. In a certain commit or snapshot, which records are inserted, which records are deleted, and which records are updated. Even for the updated records, both the old value before updated and the new value after updated are needed. + +To implement this capacity, we have to upgrade the write and read parts. Let hudi can figure out the changing data when read. And in some cases, writing the extra data to help querying the changing data if necessary. + +## Scenario Definition + +Here use a simple case to explain the CDC. + +![](scenario-definition.jpg) + +Here we follow the debezium output format, there are four columns shown: +- op: the operation of this record; +- ts_ms: the timestamp; +- source: source information such as the name of database and table. **Maybe we don't need this column in Hudi**; +- before: the previous image before this operation; +- after: the current image after this operation; + +`op` column has three enum values: +- i: represent `insert`; when `op` is `i`, `before` is always null; +- u: represent `update`; when `op` is `u`, both `before` and `after` don't be null; +- d: represent `delete`; when `op` is `d`, `after` is always null; + +Notice: +Here the illustration ignores all the metadata columns in `before` and `after` columns like `_hoodie_commit_time`. + +## Goals + +1. Support row-level CDC records generation and persistence; +2. Support both MOR and COW tables; +3. Support all the write operations; +4. Support Spark DataFrame/SQL/Streaming Query; + +## Implementation +### CDC Architecture + +![](arch.jpg) + +Notice: +Other operations like `Compact`, `Clean`, `Index` do not write/change any data. So we don't need to consider in CDC scenario. + +### Points to be upgraded + +![](points.jpg) + +### Config Definitions + +| | default | | +| --- | --- | --- | +| hoodie.table.cdf.enabled | false | `true` represents the table to be used for CDC queries and will write cdc data if needed. | Review Comment: `cdc.enabled` - today ability for incremental queries is on by default. How does the default behavior change for incremental queries? ########## rfc/rfc-51/rfc-51.md: ########## @@ -0,0 +1,234 @@ +<!-- + 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. +--> + +# RFC-50: Hudi CDC + +# Proposers + +- @Yann Byron + +# Approvers + +- @Raymond + +# Statue +JIRA: [https://issues.apache.org/jira/browse/HUDI-3478](https://issues.apache.org/jira/browse/HUDI-3478) + +# Hudi Supports Change-Data-Capture + +## Abstract + +We want to introduce the Change-Data-Capture(CDC) capacity that make hudi can produce the changeing data by which we can know how the records is changed, to response the CDC cases. + + +## Background + +In some use cases where hudi tables is used as streaing source, We want to be aware every records' changing in one commit exactly. In a certain commit or snapshot, which records are inserted, which records are deleted, and which records are updated. Even for the updated records, both the old value before updated and the new value after updated are needed. Review Comment: typo:streaming ########## rfc/rfc-51/rfc-51.md: ########## @@ -0,0 +1,234 @@ +<!-- + 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. +--> + +# RFC-50: Hudi CDC + +# Proposers + +- @Yann Byron + +# Approvers + +- @Raymond + +# Statue +JIRA: [https://issues.apache.org/jira/browse/HUDI-3478](https://issues.apache.org/jira/browse/HUDI-3478) + +# Hudi Supports Change-Data-Capture + +## Abstract + +We want to introduce the Change-Data-Capture(CDC) capacity that make hudi can produce the changeing data by which we can know how the records is changed, to response the CDC cases. Review Comment: Just for flavor, if you adopt soft deletes, you can build end-end incremental pipelines today with "new value" alone already. Just want to add context ########## rfc/rfc-51/rfc-51.md: ########## @@ -0,0 +1,234 @@ +<!-- + 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. +--> + +# RFC-50: Hudi CDC + +# Proposers + +- @Yann Byron + +# Approvers + +- @Raymond + +# Statue +JIRA: [https://issues.apache.org/jira/browse/HUDI-3478](https://issues.apache.org/jira/browse/HUDI-3478) + +# Hudi Supports Change-Data-Capture + +## Abstract + +We want to introduce the Change-Data-Capture(CDC) capacity that make hudi can produce the changeing data by which we can know how the records is changed, to response the CDC cases. Review Comment: Can we add context around incremental queries please? - this RFC fixes - hard deletes and provides a standard CDC format. But per see, we have had CDC since even first version ########## rfc/rfc-51/rfc-51.md: ########## @@ -0,0 +1,234 @@ +<!-- + 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. +--> + +# RFC-50: Hudi CDC + +# Proposers + +- @Yann Byron + +# Approvers + +- @Raymond + +# Statue +JIRA: [https://issues.apache.org/jira/browse/HUDI-3478](https://issues.apache.org/jira/browse/HUDI-3478) + +# Hudi Supports Change-Data-Capture + +## Abstract + +We want to introduce the Change-Data-Capture(CDC) capacity that make hudi can produce the changeing data by which we can know how the records is changed, to response the CDC cases. + + +## Background + +In some use cases where hudi tables is used as streaing source, We want to be aware every records' changing in one commit exactly. In a certain commit or snapshot, which records are inserted, which records are deleted, and which records are updated. Even for the updated records, both the old value before updated and the new value after updated are needed. + +To implement this capacity, we have to upgrade the write and read parts. Let hudi can figure out the changing data when read. And in some cases, writing the extra data to help querying the changing data if necessary. + +## Scenario Definition + +Here use a simple case to explain the CDC. + +![](scenario-definition.jpg) + +Here we follow the debezium output format, there are four columns shown: +- op: the operation of this record; +- ts_ms: the timestamp; +- source: source information such as the name of database and table. **Maybe we don't need this column in Hudi**; +- before: the previous image before this operation; +- after: the current image after this operation; + +`op` column has three enum values: +- i: represent `insert`; when `op` is `i`, `before` is always null; +- u: represent `update`; when `op` is `u`, both `before` and `after` don't be null; +- d: represent `delete`; when `op` is `d`, `after` is always null; + +Notice: +Here the illustration ignores all the metadata columns in `before` and `after` columns like `_hoodie_commit_time`. + +## Goals + +1. Support row-level CDC records generation and persistence; +2. Support both MOR and COW tables; +3. Support all the write operations; +4. Support Spark DataFrame/SQL/Streaming Query; + +## Implementation +### CDC Architecture + +![](arch.jpg) + +Notice: +Other operations like `Compact`, `Clean`, `Index` do not write/change any data. So we don't need to consider in CDC scenario. + +### Points to be upgraded + +![](points.jpg) + +### Config Definitions + +| | default | | +| --- | --- | --- | +| hoodie.table.cdf.enabled | false | `true` represents the table to be used for CDC queries and will write cdc data if needed. | +| | | | +| hoodie.datasource.read.cdc.enabled | false | if true, return the CDC data. | +| hoodie.datasource.read.start.timestamp | - | requried. | +| hoodie.datasource.read.end.timestamp | - | optional. | + + +### Write + +Hoodie writes data by `HoodieWriteHandle`. +We notice that only `HoodieMergeHandle` and it's subclasses will receive both the old record and the new-coming record at the same time, merge and write. +So we will add a `LogFormatWriter` in these classes. If there is CDC data need to be written out, then call this writer to write out a log file which consist of, maybe `CDCBlock`. +The CDC log file will be placed in the same position as the base files and other log files, so that the clean service can clean up them without extra work. + +The directory of the CDC file is`tablePath/.cdc/`. The file structure is like: +``` +hudi_cdc_table/ + .hoodie/ + hoodie.properties + 00001.commit + 00002.replacecommit + ... + default/ + year=2021/ + filegroup1-instant1.parquet + .filegroup1-instant1.cdc.log + year=2022/ + filegroup2-instant1.parquet + .filegroup1-instant1.cdc.log + ... +``` + +One Design Idea is that **Write CDC files as little as possible, and reuse data files as much as possible**. + +As the idea, there are four file types visible for CDC: + +- CDC File: Record all the related changing data with the cdc schema for one commit. For the following cases, will generate the CDC file: + - `UPSERT` operation; + - `DELETE` operation and the files where the data to be deleted resides has other data that doesn't need to be deleted and need to be rewrited. +- pure Add-File: all the data in this file ars incoming, and don't affect the existing data and files. In the following cases, we do not have data to be rewrited and need to write CDC data to the CDC file: Review Comment: Not sure I understand all this add-file/remove-file stuff ########## rfc/rfc-51/rfc-51.md: ########## @@ -0,0 +1,234 @@ +<!-- + 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. +--> + +# RFC-50: Hudi CDC + +# Proposers + +- @Yann Byron + +# Approvers + +- @Raymond + +# Statue +JIRA: [https://issues.apache.org/jira/browse/HUDI-3478](https://issues.apache.org/jira/browse/HUDI-3478) + +# Hudi Supports Change-Data-Capture + +## Abstract + +We want to introduce the Change-Data-Capture(CDC) capacity that make hudi can produce the changeing data by which we can know how the records is changed, to response the CDC cases. + + +## Background + +In some use cases where hudi tables is used as streaing source, We want to be aware every records' changing in one commit exactly. In a certain commit or snapshot, which records are inserted, which records are deleted, and which records are updated. Even for the updated records, both the old value before updated and the new value after updated are needed. + +To implement this capacity, we have to upgrade the write and read parts. Let hudi can figure out the changing data when read. And in some cases, writing the extra data to help querying the changing data if necessary. + +## Scenario Definition + +Here use a simple case to explain the CDC. + +![](scenario-definition.jpg) + +Here we follow the debezium output format, there are four columns shown: +- op: the operation of this record; +- ts_ms: the timestamp; +- source: source information such as the name of database and table. **Maybe we don't need this column in Hudi**; +- before: the previous image before this operation; +- after: the current image after this operation; + +`op` column has three enum values: +- i: represent `insert`; when `op` is `i`, `before` is always null; +- u: represent `update`; when `op` is `u`, both `before` and `after` don't be null; +- d: represent `delete`; when `op` is `d`, `after` is always null; + +Notice: +Here the illustration ignores all the metadata columns in `before` and `after` columns like `_hoodie_commit_time`. + +## Goals + +1. Support row-level CDC records generation and persistence; +2. Support both MOR and COW tables; +3. Support all the write operations; +4. Support Spark DataFrame/SQL/Streaming Query; Review Comment: include Flink as well? ########## rfc/rfc-51/rfc-51.md: ########## @@ -0,0 +1,234 @@ +<!-- + 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. +--> + +# RFC-50: Hudi CDC + +# Proposers + +- @Yann Byron + +# Approvers + +- @Raymond + +# Statue +JIRA: [https://issues.apache.org/jira/browse/HUDI-3478](https://issues.apache.org/jira/browse/HUDI-3478) + +# Hudi Supports Change-Data-Capture + +## Abstract + +We want to introduce the Change-Data-Capture(CDC) capacity that make hudi can produce the changeing data by which we can know how the records is changed, to response the CDC cases. + + +## Background + +In some use cases where hudi tables is used as streaing source, We want to be aware every records' changing in one commit exactly. In a certain commit or snapshot, which records are inserted, which records are deleted, and which records are updated. Even for the updated records, both the old value before updated and the new value after updated are needed. + +To implement this capacity, we have to upgrade the write and read parts. Let hudi can figure out the changing data when read. And in some cases, writing the extra data to help querying the changing data if necessary. + +## Scenario Definition + +Here use a simple case to explain the CDC. + +![](scenario-definition.jpg) + +Here we follow the debezium output format, there are four columns shown: +- op: the operation of this record; +- ts_ms: the timestamp; +- source: source information such as the name of database and table. **Maybe we don't need this column in Hudi**; +- before: the previous image before this operation; +- after: the current image after this operation; + +`op` column has three enum values: +- i: represent `insert`; when `op` is `i`, `before` is always null; +- u: represent `update`; when `op` is `u`, both `before` and `after` don't be null; +- d: represent `delete`; when `op` is `d`, `after` is always null; + +Notice: +Here the illustration ignores all the metadata columns in `before` and `after` columns like `_hoodie_commit_time`. + +## Goals + +1. Support row-level CDC records generation and persistence; +2. Support both MOR and COW tables; +3. Support all the write operations; +4. Support Spark DataFrame/SQL/Streaming Query; + +## Implementation +### CDC Architecture + +![](arch.jpg) + +Notice: +Other operations like `Compact`, `Clean`, `Index` do not write/change any data. So we don't need to consider in CDC scenario. + +### Points to be upgraded + +![](points.jpg) + +### Config Definitions + +| | default | | +| --- | --- | --- | +| hoodie.table.cdf.enabled | false | `true` represents the table to be used for CDC queries and will write cdc data if needed. | +| | | | +| hoodie.datasource.read.cdc.enabled | false | if true, return the CDC data. | +| hoodie.datasource.read.start.timestamp | - | requried. | +| hoodie.datasource.read.end.timestamp | - | optional. | + + +### Write + +Hoodie writes data by `HoodieWriteHandle`. +We notice that only `HoodieMergeHandle` and it's subclasses will receive both the old record and the new-coming record at the same time, merge and write. +So we will add a `LogFormatWriter` in these classes. If there is CDC data need to be written out, then call this writer to write out a log file which consist of, maybe `CDCBlock`. +The CDC log file will be placed in the same position as the base files and other log files, so that the clean service can clean up them without extra work. + +The directory of the CDC file is`tablePath/.cdc/`. The file structure is like: Review Comment: I thought we agreed on not writing a separate `.cdc` folder. are we writing one? -- This is an automated message from the Apache Git Service. To respond to the message, please log on to GitHub and use the URL above to go to the specific comment. To unsubscribe, e-mail: commits-unsubscr...@hudi.apache.org For queries about this service, please contact Infrastructure at: us...@infra.apache.org