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 }),
42 protocol_config: ProtocolProperties::Maxwell,
43 };
44 let mut parser = MaxwellParser::new(props, descs.clone(), SourceContext::dummy().into())
45 .await
46 .unwrap();
47
48 let mut builder = SourceStreamChunkBuilder::new(descs, SourceCtrlOpts::for_test());
49 let payloads = vec![
50 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(),
51 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(),
52 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()
53 ];
54
55 for payload in payloads {
56 parser
57 .parse_inner(payload, builder.row_writer())
58 .await
59 .unwrap();
60 }
61
62 builder.finish_current_chunk();
63 let chunk = builder.consume_ready_chunks().next().unwrap();
64
65 let mut rows = chunk.rows();
66
67 {
68 let (op, row) = rows.next().unwrap();
69 assert_eq!(op, Op::Insert);
70 assert_eq!(row.datum_at(0).to_owned_datum(), Some(ScalarImpl::Int32(1)));
71 assert_eq!(
72 row.datum_at(1).to_owned_datum(),
73 (Some(ScalarImpl::Utf8("tom".into())))
74 );
75 assert_eq!(
76 row.datum_at(2).to_owned_datum(),
77 (Some(ScalarImpl::Int16(0)))
78 );
79 assert_eq!(
80 row.datum_at(3).to_owned_datum(),
81 (Some(ScalarImpl::Timestamp(
82 "2017-12-31 16:00:01".parse().unwrap()
83 )))
84 )
85 }
86
87 {
88 let (op, row) = rows.next().unwrap();
89 assert_eq!(op, Op::Insert);
90 assert_eq!(row.datum_at(0).to_owned_datum(), Some(ScalarImpl::Int32(2)));
91 assert_eq!(
92 row.datum_at(1).to_owned_datum(),
93 (Some(ScalarImpl::Utf8("alex".into())))
94 );
95 assert_eq!(
96 row.datum_at(2).to_owned_datum(),
97 (Some(ScalarImpl::Int16(1)))
98 );
99 assert_eq!(
100 row.datum_at(3).to_owned_datum(),
101 (Some(ScalarImpl::Timestamp(
102 "1999-12-31 16:00:01".parse().unwrap()
103 )))
104 )
105 }
106
107 {
108 let (op, row) = rows.next().unwrap();
109 assert_eq!(op, Op::Insert);
110 assert_eq!(row.datum_at(0).to_owned_datum(), Some(ScalarImpl::Int32(2)));
111 assert_eq!(
112 row.datum_at(1).to_owned_datum(),
113 (Some(ScalarImpl::Utf8("chi".into())))
114 );
115 assert_eq!(
116 row.datum_at(2).to_owned_datum(),
117 (Some(ScalarImpl::Int16(1)))
118 );
119 assert_eq!(
120 row.datum_at(3).to_owned_datum(),
121 (Some(ScalarImpl::Timestamp(
122 "1999-12-31 16:00:01".parse().unwrap()
123 )))
124 )
125 }
126 }
127}