risingwave_connector/parser/maxwell/
simd_json_parser.rs1#[cfg(test)]
16mod tests {
17 use risingwave_common::array::Op;
18 use risingwave_common::row::Row;
19 use risingwave_common::types::{DataType, ScalarImpl, ToOwnedDatum};
20
21 use crate::parser::maxwell::MaxwellParser;
22 use crate::parser::{
23 EncodingProperties, JsonProperties, ProtocolProperties, SourceColumnDesc,
24 SourceStreamChunkBuilder, SpecificParserConfig,
25 };
26 use crate::source::{SourceContext, SourceCtrlOpts};
27
28 #[tokio::test]
29 async fn test_json_parser() {
30 let descs = vec![
31 SourceColumnDesc::simple("ID", DataType::Int32, 0.into()),
32 SourceColumnDesc::simple("NAME", DataType::Varchar, 1.into()),
33 SourceColumnDesc::simple("is_adult", DataType::Int16, 2.into()),
34 SourceColumnDesc::simple("birthday", DataType::Timestamp, 3.into()),
35 ];
36
37 let props = SpecificParserConfig {
38 encoding_config: EncodingProperties::Json(JsonProperties {
39 use_schema_registry: false,
40 timestamptz_handling: None,
41 timestamp_handling: None,
42 time_handling: None,
43 handle_toast_columns: false,
44 }),
45 protocol_config: ProtocolProperties::Maxwell,
46 };
47 let mut parser = MaxwellParser::new(props, descs.clone(), SourceContext::dummy().into())
48 .await
49 .unwrap();
50
51 let mut builder = SourceStreamChunkBuilder::new(descs, SourceCtrlOpts::for_test());
52 let payloads = vec![
53 br#"{"database":"test","table":"t","type":"insert","ts":1666937996,"xid":1171,"commit":true,"data":{"id":1,"name":"tom","is_adult":0,"birthday":"2017-12-31 16:00:01"}}"#.to_vec(),
54 br#"{"database":"test","table":"t","type":"insert","ts":1666938023,"xid":1254,"commit":true,"data":{"id":2,"name":"alex","is_adult":1,"birthday":"1999-12-31 16:00:01"}}"#.to_vec(),
55 br#"{"database":"test","table":"t","type":"update","ts":1666938068,"xid":1373,"commit":true,"data":{"id":2,"name":"chi","is_adult":1,"birthday":"1999-12-31 16:00:01"},"old":{"name":"alex"}}"#.to_vec()
56 ];
57
58 for payload in payloads {
59 parser
60 .parse_inner(payload, builder.row_writer())
61 .await
62 .unwrap();
63 }
64
65 builder.finish_current_chunk();
66 let chunk = builder.consume_ready_chunks().next().unwrap();
67
68 let mut rows = chunk.rows();
69
70 {
71 let (op, row) = rows.next().unwrap();
72 assert_eq!(op, Op::Insert);
73 assert_eq!(row.datum_at(0).to_owned_datum(), Some(ScalarImpl::Int32(1)));
74 assert_eq!(
75 row.datum_at(1).to_owned_datum(),
76 (Some(ScalarImpl::Utf8("tom".into())))
77 );
78 assert_eq!(
79 row.datum_at(2).to_owned_datum(),
80 (Some(ScalarImpl::Int16(0)))
81 );
82 assert_eq!(
83 row.datum_at(3).to_owned_datum(),
84 (Some(ScalarImpl::Timestamp(
85 "2017-12-31 16:00:01".parse().unwrap()
86 )))
87 )
88 }
89
90 {
91 let (op, row) = rows.next().unwrap();
92 assert_eq!(op, Op::Insert);
93 assert_eq!(row.datum_at(0).to_owned_datum(), Some(ScalarImpl::Int32(2)));
94 assert_eq!(
95 row.datum_at(1).to_owned_datum(),
96 (Some(ScalarImpl::Utf8("alex".into())))
97 );
98 assert_eq!(
99 row.datum_at(2).to_owned_datum(),
100 (Some(ScalarImpl::Int16(1)))
101 );
102 assert_eq!(
103 row.datum_at(3).to_owned_datum(),
104 (Some(ScalarImpl::Timestamp(
105 "1999-12-31 16:00:01".parse().unwrap()
106 )))
107 )
108 }
109
110 {
111 let (op, row) = rows.next().unwrap();
112 assert_eq!(op, Op::Insert);
113 assert_eq!(row.datum_at(0).to_owned_datum(), Some(ScalarImpl::Int32(2)));
114 assert_eq!(
115 row.datum_at(1).to_owned_datum(),
116 (Some(ScalarImpl::Utf8("chi".into())))
117 );
118 assert_eq!(
119 row.datum_at(2).to_owned_datum(),
120 (Some(ScalarImpl::Int16(1)))
121 );
122 assert_eq!(
123 row.datum_at(3).to_owned_datum(),
124 (Some(ScalarImpl::Timestamp(
125 "1999-12-31 16:00:01".parse().unwrap()
126 )))
127 )
128 }
129 }
130}