planner_test.go 92 KB

123456789101112131415161718192021222324252627282930313233343536373839404142434445464748495051525354555657585960616263646566676869707172737475767778798081828384858687888990919293949596979899100101102103104105106107108109110111112113114115116117118119120121122123124125126127128129130131132133134135136137138139140141142143144145146147148149150151152153154155156157158159160161162163164165166167168169170171172173174175176177178179180181182183184185186187188189190191192193194195196197198199200201202203204205206207208209210211212213214215216217218219220221222223224225226227228229230231232233234235236237238239240241242243244245246247248249250251252253254255256257258259260261262263264265266267268269270271272273274275276277278279280281282283284285286287288289290291292293294295296297298299300301302303304305306307308309310311312313314315316317318319320321322323324325326327328329330331332333334335336337338339340341342343344345346347348349350351352353354355356357358359360361362363364365366367368369370371372373374375376377378379380381382383384385386387388389390391392393394395396397398399400401402403404405406407408409410411412413414415416417418419420421422423424425426427428429430431432433434435436437438439440441442443444445446447448449450451452453454455456457458459460461462463464465466467468469470471472473474475476477478479480481482483484485486487488489490491492493494495496497498499500501502503504505506507508509510511512513514515516517518519520521522523524525526527528529530531532533534535536537538539540541542543544545546547548549550551552553554555556557558559560561562563564565566567568569570571572573574575576577578579580581582583584585586587588589590591592593594595596597598599600601602603604605606607608609610611612613614615616617618619620621622623624625626627628629630631632633634635636637638639640641642643644645646647648649650651652653654655656657658659660661662663664665666667668669670671672673674675676677678679680681682683684685686687688689690691692693694695696697698699700701702703704705706707708709710711712713714715716717718719720721722723724725726727728729730731732733734735736737738739740741742743744745746747748749750751752753754755756757758759760761762763764765766767768769770771772773774775776777778779780781782783784785786787788789790791792793794795796797798799800801802803804805806807808809810811812813814815816817818819820821822823824825826827828829830831832833834835836837838839840841842843844845846847848849850851852853854855856857858859860861862863864865866867868869870871872873874875876877878879880881882883884885886887888889890891892893894895896897898899900901902903904905906907908909910911912913914915916917918919920921922923924925926927928929930931932933934935936937938939940941942943944945946947948949950951952953954955956957958959960961962963964965966967968969970971972973974975976977978979980981982983984985986987988989990991992993994995996997998999100010011002100310041005100610071008100910101011101210131014101510161017101810191020102110221023102410251026102710281029103010311032103310341035103610371038103910401041104210431044104510461047104810491050105110521053105410551056105710581059106010611062106310641065106610671068106910701071107210731074107510761077107810791080108110821083108410851086108710881089109010911092109310941095109610971098109911001101110211031104110511061107110811091110111111121113111411151116111711181119112011211122112311241125112611271128112911301131113211331134113511361137113811391140114111421143114411451146114711481149115011511152115311541155115611571158115911601161116211631164116511661167116811691170117111721173117411751176117711781179118011811182118311841185118611871188118911901191119211931194119511961197119811991200120112021203120412051206120712081209121012111212121312141215121612171218121912201221122212231224122512261227122812291230123112321233123412351236123712381239124012411242124312441245124612471248124912501251125212531254125512561257125812591260126112621263126412651266126712681269127012711272127312741275127612771278127912801281128212831284128512861287128812891290129112921293129412951296129712981299130013011302130313041305130613071308130913101311131213131314131513161317131813191320132113221323132413251326132713281329133013311332133313341335133613371338133913401341134213431344134513461347134813491350135113521353135413551356135713581359136013611362136313641365136613671368136913701371137213731374137513761377137813791380138113821383138413851386138713881389139013911392139313941395139613971398139914001401140214031404140514061407140814091410141114121413141414151416141714181419142014211422142314241425142614271428142914301431143214331434143514361437143814391440144114421443144414451446144714481449145014511452145314541455145614571458145914601461146214631464146514661467146814691470147114721473147414751476147714781479148014811482148314841485148614871488148914901491149214931494149514961497149814991500150115021503150415051506150715081509151015111512151315141515151615171518151915201521152215231524152515261527152815291530153115321533153415351536153715381539154015411542154315441545154615471548154915501551155215531554155515561557155815591560156115621563156415651566156715681569157015711572157315741575157615771578157915801581158215831584158515861587158815891590159115921593159415951596159715981599160016011602160316041605160616071608160916101611161216131614161516161617161816191620162116221623162416251626162716281629163016311632163316341635163616371638163916401641164216431644164516461647164816491650165116521653165416551656165716581659166016611662166316641665166616671668166916701671167216731674167516761677167816791680168116821683168416851686168716881689169016911692169316941695169616971698169917001701170217031704170517061707170817091710171117121713171417151716171717181719172017211722172317241725172617271728172917301731173217331734173517361737173817391740174117421743174417451746174717481749175017511752175317541755175617571758175917601761176217631764176517661767176817691770177117721773177417751776177717781779178017811782178317841785178617871788178917901791179217931794179517961797179817991800180118021803180418051806180718081809181018111812181318141815181618171818181918201821182218231824182518261827182818291830183118321833183418351836183718381839184018411842184318441845184618471848184918501851185218531854185518561857185818591860186118621863186418651866186718681869187018711872187318741875187618771878187918801881188218831884188518861887188818891890189118921893189418951896189718981899190019011902190319041905190619071908190919101911191219131914191519161917191819191920192119221923192419251926192719281929193019311932193319341935193619371938193919401941194219431944194519461947194819491950195119521953195419551956195719581959196019611962196319641965196619671968196919701971197219731974197519761977197819791980198119821983198419851986198719881989199019911992199319941995199619971998199920002001200220032004200520062007200820092010201120122013201420152016201720182019202020212022202320242025202620272028202920302031203220332034203520362037203820392040204120422043204420452046204720482049205020512052205320542055205620572058205920602061206220632064206520662067206820692070207120722073207420752076207720782079208020812082208320842085208620872088208920902091209220932094209520962097209820992100210121022103210421052106210721082109211021112112211321142115211621172118211921202121212221232124212521262127212821292130213121322133213421352136213721382139214021412142214321442145214621472148214921502151215221532154215521562157215821592160216121622163216421652166216721682169217021712172217321742175217621772178217921802181218221832184218521862187218821892190219121922193219421952196219721982199220022012202220322042205220622072208220922102211221222132214221522162217221822192220222122222223222422252226222722282229223022312232223322342235223622372238223922402241224222432244224522462247224822492250225122522253225422552256225722582259226022612262226322642265226622672268226922702271227222732274227522762277227822792280228122822283228422852286228722882289229022912292229322942295229622972298229923002301230223032304230523062307230823092310231123122313231423152316231723182319232023212322232323242325232623272328232923302331233223332334233523362337233823392340234123422343234423452346234723482349235023512352235323542355235623572358235923602361236223632364236523662367236823692370237123722373237423752376237723782379238023812382238323842385238623872388238923902391239223932394239523962397239823992400240124022403240424052406240724082409241024112412241324142415241624172418241924202421242224232424242524262427242824292430243124322433243424352436243724382439244024412442244324442445244624472448244924502451245224532454245524562457245824592460246124622463246424652466246724682469247024712472247324742475247624772478247924802481248224832484248524862487248824892490249124922493249424952496249724982499250025012502250325042505250625072508250925102511251225132514251525162517251825192520252125222523252425252526252725282529253025312532253325342535253625372538253925402541254225432544254525462547254825492550255125522553255425552556255725582559256025612562256325642565256625672568256925702571257225732574257525762577257825792580258125822583258425852586258725882589259025912592259325942595259625972598259926002601260226032604260526062607260826092610261126122613261426152616261726182619262026212622262326242625262626272628262926302631263226332634263526362637263826392640264126422643264426452646264726482649265026512652265326542655265626572658265926602661266226632664266526662667266826692670267126722673267426752676267726782679268026812682268326842685268626872688268926902691269226932694269526962697269826992700270127022703270427052706270727082709271027112712271327142715271627172718271927202721272227232724272527262727272827292730273127322733273427352736273727382739274027412742274327442745274627472748274927502751275227532754275527562757275827592760276127622763276427652766276727682769277027712772277327742775277627772778277927802781278227832784278527862787278827892790279127922793279427952796279727982799280028012802280328042805280628072808280928102811281228132814281528162817281828192820282128222823282428252826282728282829283028312832283328342835283628372838283928402841284228432844284528462847284828492850285128522853285428552856285728582859286028612862286328642865286628672868286928702871287228732874287528762877287828792880288128822883288428852886288728882889289028912892289328942895289628972898289929002901290229032904290529062907290829092910291129122913291429152916291729182919292029212922292329242925292629272928292929302931293229332934293529362937293829392940294129422943294429452946294729482949295029512952295329542955295629572958295929602961296229632964296529662967296829692970297129722973297429752976297729782979298029812982298329842985298629872988298929902991299229932994299529962997299829993000300130023003300430053006300730083009301030113012301330143015301630173018301930203021302230233024302530263027302830293030303130323033303430353036303730383039304030413042304330443045304630473048304930503051305230533054305530563057305830593060306130623063306430653066306730683069307030713072307330743075307630773078307930803081308230833084308530863087308830893090309130923093309430953096309730983099310031013102310331043105310631073108310931103111311231133114311531163117311831193120312131223123312431253126312731283129313031313132313331343135313631373138313931403141314231433144314531463147314831493150315131523153315431553156315731583159316031613162316331643165316631673168316931703171317231733174317531763177317831793180318131823183318431853186318731883189319031913192319331943195319631973198319932003201320232033204320532063207320832093210321132123213321432153216321732183219322032213222322332243225322632273228322932303231323232333234323532363237323832393240324132423243324432453246324732483249325032513252325332543255325632573258325932603261326232633264326532663267326832693270327132723273327432753276327732783279328032813282328332843285328632873288328932903291329232933294329532963297329832993300330133023303330433053306330733083309331033113312331333143315331633173318331933203321332233233324332533263327332833293330333133323333333433353336333733383339334033413342334333443345334633473348334933503351335233533354
  1. // Copyright 2022-2023 EMQ Technologies Co., Ltd.
  2. //
  3. // Licensed under the Apache License, Version 2.0 (the "License");
  4. // you may not use this file except in compliance with the License.
  5. // You may obtain a copy of the License at
  6. //
  7. // http://www.apache.org/licenses/LICENSE-2.0
  8. //
  9. // Unless required by applicable law or agreed to in writing, software
  10. // distributed under the License is distributed on an "AS IS" BASIS,
  11. // WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
  12. // See the License for the specific language governing permissions and
  13. // limitations under the License.
  14. package planner
  15. import (
  16. "encoding/json"
  17. "fmt"
  18. "reflect"
  19. "strings"
  20. "testing"
  21. "github.com/gdexlab/go-render/render"
  22. "github.com/lf-edge/ekuiper/internal/pkg/store"
  23. "github.com/lf-edge/ekuiper/internal/testx"
  24. "github.com/lf-edge/ekuiper/internal/topo/node"
  25. "github.com/lf-edge/ekuiper/internal/xsql"
  26. "github.com/lf-edge/ekuiper/pkg/api"
  27. "github.com/lf-edge/ekuiper/pkg/ast"
  28. )
  29. func init() {
  30. testx.InitEnv()
  31. }
  32. func Test_createLogicalPlan(t *testing.T) {
  33. kv, err := store.GetKV("stream")
  34. if err != nil {
  35. t.Error(err)
  36. return
  37. }
  38. streamSqls := map[string]string{
  39. "src1": `CREATE STREAM src1 (
  40. id1 BIGINT,
  41. temp BIGINT,
  42. name string,
  43. myarray array(string)
  44. ) WITH (DATASOURCE="src1", FORMAT="json", KEY="ts");`,
  45. "src2": `CREATE STREAM src2 (
  46. id2 BIGINT,
  47. hum BIGINT
  48. ) WITH (DATASOURCE="src2", FORMAT="json", KEY="ts", TIMESTAMP_FORMAT="YYYY-MM-dd HH:mm:ss");`,
  49. "tableInPlanner": `CREATE TABLE tableInPlanner (
  50. id BIGINT,
  51. name STRING,
  52. value STRING,
  53. hum BIGINT
  54. ) WITH (TYPE="file");`,
  55. }
  56. types := map[string]ast.StreamType{
  57. "src1": ast.TypeStream,
  58. "src2": ast.TypeStream,
  59. "tableInPlanner": ast.TypeTable,
  60. }
  61. for name, sql := range streamSqls {
  62. s, err := json.Marshal(&xsql.StreamInfo{
  63. StreamType: types[name],
  64. Statement: sql,
  65. })
  66. if err != nil {
  67. t.Error(err)
  68. t.Fail()
  69. }
  70. err = kv.Set(name, string(s))
  71. if err != nil {
  72. t.Error(err)
  73. t.Fail()
  74. }
  75. }
  76. streams := make(map[string]*ast.StreamStmt)
  77. for n := range streamSqls {
  78. streamStmt, err := xsql.GetDataSource(kv, n)
  79. if err != nil {
  80. t.Errorf("fail to get stream %s, please check if stream is created", n)
  81. return
  82. }
  83. streams[n] = streamStmt
  84. }
  85. // boolTrue = true
  86. boolFalse := false
  87. tests := []struct {
  88. sql string
  89. p LogicalPlan
  90. err string
  91. }{
  92. {
  93. sql: "select unnest(myarray) as col from src1",
  94. p: ProjectSetPlan{
  95. SrfMapping: map[string]struct{}{
  96. "col": {},
  97. },
  98. baseLogicalPlan: baseLogicalPlan{
  99. children: []LogicalPlan{
  100. ProjectPlan{
  101. baseLogicalPlan: baseLogicalPlan{
  102. children: []LogicalPlan{
  103. DataSourcePlan{
  104. baseLogicalPlan: baseLogicalPlan{},
  105. name: "src1",
  106. streamFields: map[string]*ast.JsonStreamField{
  107. "myarray": {
  108. Type: "array",
  109. Items: &ast.JsonStreamField{
  110. Type: "string",
  111. },
  112. },
  113. },
  114. streamStmt: streams["src1"],
  115. metaFields: []string{},
  116. }.Init(),
  117. },
  118. },
  119. fields: []ast.Field{
  120. {
  121. Name: "unnest",
  122. AName: "col",
  123. Expr: func() *ast.FieldRef {
  124. fr := &ast.FieldRef{
  125. StreamName: ast.AliasStream,
  126. Name: "col",
  127. AliasRef: &ast.AliasRef{
  128. Expression: &ast.Call{
  129. Name: "unnest",
  130. FuncType: ast.FuncTypeSrf,
  131. Args: []ast.Expr{
  132. &ast.FieldRef{
  133. StreamName: "src1",
  134. Name: "myarray",
  135. },
  136. },
  137. },
  138. },
  139. }
  140. fr.SetRefSource([]ast.StreamName{"src1"})
  141. return fr
  142. }(),
  143. },
  144. },
  145. }.Init(),
  146. },
  147. },
  148. }.Init(),
  149. },
  150. { // 0
  151. sql: "SELECT unnest(myarray), name from src1",
  152. p: ProjectSetPlan{
  153. SrfMapping: map[string]struct{}{
  154. "unnest": {},
  155. },
  156. baseLogicalPlan: baseLogicalPlan{
  157. children: []LogicalPlan{
  158. ProjectPlan{
  159. baseLogicalPlan: baseLogicalPlan{
  160. children: []LogicalPlan{
  161. DataSourcePlan{
  162. baseLogicalPlan: baseLogicalPlan{},
  163. name: "src1",
  164. streamFields: map[string]*ast.JsonStreamField{
  165. "myarray": {
  166. Type: "array",
  167. Items: &ast.JsonStreamField{
  168. Type: "string",
  169. },
  170. },
  171. "name": {
  172. Type: "string",
  173. },
  174. },
  175. streamStmt: streams["src1"],
  176. metaFields: []string{},
  177. }.Init(),
  178. },
  179. },
  180. fields: []ast.Field{
  181. {
  182. Expr: &ast.Call{
  183. Name: "unnest",
  184. FuncType: ast.FuncTypeSrf,
  185. Args: []ast.Expr{
  186. &ast.FieldRef{
  187. StreamName: "src1",
  188. Name: "myarray",
  189. },
  190. },
  191. },
  192. Name: "unnest",
  193. },
  194. {
  195. Name: "name",
  196. Expr: &ast.FieldRef{
  197. StreamName: "src1",
  198. Name: "name",
  199. },
  200. },
  201. },
  202. }.Init(),
  203. },
  204. },
  205. }.Init(),
  206. },
  207. { // 0
  208. sql: `SELECT myarray[temp] FROM src1`,
  209. p: ProjectPlan{
  210. baseLogicalPlan: baseLogicalPlan{
  211. children: []LogicalPlan{
  212. DataSourcePlan{
  213. baseLogicalPlan: baseLogicalPlan{},
  214. name: "src1",
  215. streamFields: map[string]*ast.JsonStreamField{
  216. "myarray": {
  217. Type: "array",
  218. Items: &ast.JsonStreamField{
  219. Type: "string",
  220. },
  221. },
  222. "temp": {
  223. Type: "bigint",
  224. },
  225. },
  226. streamStmt: streams["src1"],
  227. metaFields: []string{},
  228. }.Init(),
  229. },
  230. },
  231. fields: []ast.Field{
  232. {
  233. Expr: &ast.BinaryExpr{
  234. OP: ast.SUBSET,
  235. LHS: &ast.FieldRef{
  236. StreamName: "src1",
  237. Name: "myarray",
  238. },
  239. RHS: &ast.IndexExpr{Index: &ast.FieldRef{
  240. StreamName: "src1",
  241. Name: "temp",
  242. }},
  243. },
  244. Name: "kuiper_field_0",
  245. AName: "",
  246. },
  247. },
  248. isAggregate: false,
  249. sendMeta: false,
  250. }.Init(),
  251. },
  252. { // 1 optimize where to data source
  253. sql: `SELECT temp FROM src1 WHERE name = "v1" GROUP BY TUMBLINGWINDOW(ss, 10)`,
  254. p: ProjectPlan{
  255. baseLogicalPlan: baseLogicalPlan{
  256. children: []LogicalPlan{
  257. WindowPlan{
  258. baseLogicalPlan: baseLogicalPlan{
  259. children: []LogicalPlan{
  260. FilterPlan{
  261. baseLogicalPlan: baseLogicalPlan{
  262. children: []LogicalPlan{
  263. DataSourcePlan{
  264. name: "src1",
  265. streamFields: map[string]*ast.JsonStreamField{
  266. "name": {
  267. Type: "string",
  268. },
  269. "temp": {
  270. Type: "bigint",
  271. },
  272. },
  273. streamStmt: streams["src1"],
  274. metaFields: []string{},
  275. }.Init(),
  276. },
  277. },
  278. condition: &ast.BinaryExpr{
  279. LHS: &ast.FieldRef{Name: "name", StreamName: "src1"},
  280. OP: ast.EQ,
  281. RHS: &ast.StringLiteral{Val: "v1"},
  282. },
  283. }.Init(),
  284. },
  285. },
  286. condition: nil,
  287. wtype: ast.TUMBLING_WINDOW,
  288. length: 10,
  289. timeUnit: ast.SS,
  290. interval: 0,
  291. limit: 0,
  292. }.Init(),
  293. },
  294. },
  295. fields: []ast.Field{
  296. {
  297. Expr: &ast.FieldRef{Name: "temp", StreamName: "src1"},
  298. Name: "temp",
  299. AName: "",
  300. },
  301. },
  302. isAggregate: false,
  303. sendMeta: false,
  304. }.Init(),
  305. },
  306. { // 2 condition that cannot be optimized
  307. sql: `SELECT id1 FROM src1 INNER JOIN src2 on src1.id1 = src2.id2 WHERE src1.temp > 20 OR src2.hum > 60 GROUP BY TUMBLINGWINDOW(ss, 10)`,
  308. p: ProjectPlan{
  309. baseLogicalPlan: baseLogicalPlan{
  310. children: []LogicalPlan{
  311. JoinPlan{
  312. baseLogicalPlan: baseLogicalPlan{
  313. children: []LogicalPlan{
  314. WindowPlan{
  315. baseLogicalPlan: baseLogicalPlan{
  316. children: []LogicalPlan{
  317. DataSourcePlan{
  318. name: "src1",
  319. streamFields: map[string]*ast.JsonStreamField{
  320. "id1": {
  321. Type: "bigint",
  322. },
  323. "temp": {
  324. Type: "bigint",
  325. },
  326. },
  327. streamStmt: streams["src1"],
  328. metaFields: []string{},
  329. }.Init(),
  330. DataSourcePlan{
  331. name: "src2",
  332. streamFields: map[string]*ast.JsonStreamField{
  333. "hum": {
  334. Type: "bigint",
  335. },
  336. "id2": {
  337. Type: "bigint",
  338. },
  339. },
  340. streamStmt: streams["src2"],
  341. metaFields: []string{},
  342. timestampFormat: "YYYY-MM-dd HH:mm:ss",
  343. }.Init(),
  344. },
  345. },
  346. condition: nil,
  347. wtype: ast.TUMBLING_WINDOW,
  348. length: 10,
  349. timeUnit: ast.SS,
  350. interval: 0,
  351. limit: 0,
  352. }.Init(),
  353. },
  354. },
  355. from: &ast.Table{Name: "src1"},
  356. joins: ast.Joins{ast.Join{
  357. Name: "src2",
  358. JoinType: ast.INNER_JOIN,
  359. Expr: &ast.BinaryExpr{
  360. OP: ast.AND,
  361. LHS: &ast.BinaryExpr{
  362. LHS: &ast.BinaryExpr{
  363. OP: ast.GT,
  364. LHS: &ast.FieldRef{Name: "temp", StreamName: "src1"},
  365. RHS: &ast.IntegerLiteral{Val: 20},
  366. },
  367. OP: ast.OR,
  368. RHS: &ast.BinaryExpr{
  369. OP: ast.GT,
  370. LHS: &ast.FieldRef{Name: "hum", StreamName: "src2"},
  371. RHS: &ast.IntegerLiteral{Val: 60},
  372. },
  373. },
  374. RHS: &ast.BinaryExpr{
  375. OP: ast.EQ,
  376. LHS: &ast.FieldRef{Name: "id1", StreamName: "src1"},
  377. RHS: &ast.FieldRef{Name: "id2", StreamName: "src2"},
  378. },
  379. },
  380. }},
  381. }.Init(),
  382. },
  383. },
  384. fields: []ast.Field{
  385. {
  386. Expr: &ast.FieldRef{Name: "id1", StreamName: "src1"},
  387. Name: "id1",
  388. AName: "",
  389. },
  390. },
  391. isAggregate: false,
  392. sendMeta: false,
  393. }.Init(),
  394. },
  395. { // 3 optimize window filter
  396. sql: `SELECT id1 FROM src1 WHERE name = "v1" GROUP BY TUMBLINGWINDOW(ss, 10) FILTER( WHERE temp > 2)`,
  397. p: ProjectPlan{
  398. baseLogicalPlan: baseLogicalPlan{
  399. children: []LogicalPlan{
  400. WindowPlan{
  401. baseLogicalPlan: baseLogicalPlan{
  402. children: []LogicalPlan{
  403. FilterPlan{
  404. baseLogicalPlan: baseLogicalPlan{
  405. children: []LogicalPlan{
  406. DataSourcePlan{
  407. name: "src1",
  408. streamFields: map[string]*ast.JsonStreamField{
  409. "id1": {
  410. Type: "bigint",
  411. },
  412. "name": {
  413. Type: "string",
  414. },
  415. "temp": {
  416. Type: "bigint",
  417. },
  418. },
  419. streamStmt: streams["src1"],
  420. metaFields: []string{},
  421. }.Init(),
  422. },
  423. },
  424. condition: &ast.BinaryExpr{
  425. OP: ast.AND,
  426. LHS: &ast.BinaryExpr{
  427. LHS: &ast.FieldRef{Name: "name", StreamName: "src1"},
  428. OP: ast.EQ,
  429. RHS: &ast.StringLiteral{Val: "v1"},
  430. },
  431. RHS: &ast.BinaryExpr{
  432. LHS: &ast.FieldRef{Name: "temp", StreamName: "src1"},
  433. OP: ast.GT,
  434. RHS: &ast.IntegerLiteral{Val: 2},
  435. },
  436. },
  437. }.Init(),
  438. },
  439. },
  440. condition: nil,
  441. wtype: ast.TUMBLING_WINDOW,
  442. length: 10,
  443. timeUnit: ast.SS,
  444. interval: 0,
  445. limit: 0,
  446. }.Init(),
  447. },
  448. },
  449. fields: []ast.Field{
  450. {
  451. Expr: &ast.FieldRef{Name: "id1", StreamName: "src1"},
  452. Name: "id1",
  453. AName: "",
  454. },
  455. },
  456. isAggregate: false,
  457. sendMeta: false,
  458. }.Init(),
  459. },
  460. { // 4. do not optimize count window
  461. sql: `SELECT * FROM src1 WHERE temp > 20 GROUP BY COUNTWINDOW(5,1) HAVING COUNT(*) > 2`,
  462. p: ProjectPlan{
  463. baseLogicalPlan: baseLogicalPlan{
  464. children: []LogicalPlan{
  465. HavingPlan{
  466. baseLogicalPlan: baseLogicalPlan{
  467. children: []LogicalPlan{
  468. FilterPlan{
  469. baseLogicalPlan: baseLogicalPlan{
  470. children: []LogicalPlan{
  471. WindowPlan{
  472. baseLogicalPlan: baseLogicalPlan{
  473. children: []LogicalPlan{
  474. DataSourcePlan{
  475. name: "src1",
  476. isWildCard: true,
  477. streamFields: map[string]*ast.JsonStreamField{
  478. "id1": {
  479. Type: "bigint",
  480. },
  481. "temp": {
  482. Type: "bigint",
  483. },
  484. "name": {
  485. Type: "string",
  486. },
  487. "myarray": {
  488. Type: "array",
  489. Items: &ast.JsonStreamField{
  490. Type: "string",
  491. },
  492. },
  493. },
  494. streamStmt: streams["src1"],
  495. metaFields: []string{},
  496. }.Init(),
  497. },
  498. },
  499. condition: nil,
  500. wtype: ast.COUNT_WINDOW,
  501. length: 5,
  502. interval: 1,
  503. limit: 0,
  504. }.Init(),
  505. },
  506. },
  507. condition: &ast.BinaryExpr{
  508. LHS: &ast.FieldRef{Name: "temp", StreamName: "src1"},
  509. OP: ast.GT,
  510. RHS: &ast.IntegerLiteral{Val: 20},
  511. },
  512. }.Init(),
  513. },
  514. },
  515. condition: &ast.BinaryExpr{
  516. LHS: &ast.Call{Name: "count", FuncId: 0, Args: []ast.Expr{&ast.Wildcard{
  517. Token: ast.ASTERISK,
  518. }}, FuncType: ast.FuncTypeAgg},
  519. OP: ast.GT,
  520. RHS: &ast.IntegerLiteral{Val: 2},
  521. },
  522. }.Init(),
  523. },
  524. },
  525. fields: []ast.Field{
  526. {
  527. Expr: &ast.Wildcard{Token: ast.ASTERISK},
  528. Name: "*",
  529. AName: "",
  530. },
  531. },
  532. isAggregate: false,
  533. sendMeta: false,
  534. }.Init(),
  535. },
  536. { // 5. optimize join on
  537. sql: `SELECT id1 FROM src1 INNER JOIN src2 on src1.id1 = src2.id2 and src1.temp > 20 and src2.hum < 60 WHERE src1.id1 > 111 GROUP BY TUMBLINGWINDOW(ss, 10)`,
  538. p: ProjectPlan{
  539. baseLogicalPlan: baseLogicalPlan{
  540. children: []LogicalPlan{
  541. JoinPlan{
  542. baseLogicalPlan: baseLogicalPlan{
  543. children: []LogicalPlan{
  544. WindowPlan{
  545. baseLogicalPlan: baseLogicalPlan{
  546. children: []LogicalPlan{
  547. FilterPlan{
  548. baseLogicalPlan: baseLogicalPlan{
  549. children: []LogicalPlan{
  550. DataSourcePlan{
  551. name: "src1",
  552. streamFields: map[string]*ast.JsonStreamField{
  553. "id1": {
  554. Type: "bigint",
  555. },
  556. "temp": {
  557. Type: "bigint",
  558. },
  559. },
  560. streamStmt: streams["src1"],
  561. metaFields: []string{},
  562. }.Init(),
  563. },
  564. },
  565. condition: &ast.BinaryExpr{
  566. RHS: &ast.BinaryExpr{
  567. OP: ast.GT,
  568. LHS: &ast.FieldRef{Name: "temp", StreamName: "src1"},
  569. RHS: &ast.IntegerLiteral{Val: 20},
  570. },
  571. OP: ast.AND,
  572. LHS: &ast.BinaryExpr{
  573. OP: ast.GT,
  574. LHS: &ast.FieldRef{Name: "id1", StreamName: "src1"},
  575. RHS: &ast.IntegerLiteral{Val: 111},
  576. },
  577. },
  578. }.Init(),
  579. FilterPlan{
  580. baseLogicalPlan: baseLogicalPlan{
  581. children: []LogicalPlan{
  582. DataSourcePlan{
  583. name: "src2",
  584. streamFields: map[string]*ast.JsonStreamField{
  585. "hum": {
  586. Type: "bigint",
  587. },
  588. "id2": {
  589. Type: "bigint",
  590. },
  591. },
  592. streamStmt: streams["src2"],
  593. metaFields: []string{},
  594. timestampFormat: "YYYY-MM-dd HH:mm:ss",
  595. }.Init(),
  596. },
  597. },
  598. condition: &ast.BinaryExpr{
  599. OP: ast.LT,
  600. LHS: &ast.FieldRef{Name: "hum", StreamName: "src2"},
  601. RHS: &ast.IntegerLiteral{Val: 60},
  602. },
  603. }.Init(),
  604. },
  605. },
  606. condition: nil,
  607. wtype: ast.TUMBLING_WINDOW,
  608. length: 10,
  609. timeUnit: ast.SS,
  610. interval: 0,
  611. limit: 0,
  612. }.Init(),
  613. },
  614. },
  615. from: &ast.Table{
  616. Name: "src1",
  617. },
  618. joins: []ast.Join{
  619. {
  620. Name: "src2",
  621. Alias: "",
  622. JoinType: ast.INNER_JOIN,
  623. Expr: &ast.BinaryExpr{
  624. LHS: &ast.FieldRef{Name: "id1", StreamName: "src1"},
  625. OP: ast.EQ,
  626. RHS: &ast.FieldRef{Name: "id2", StreamName: "src2"},
  627. },
  628. },
  629. },
  630. }.Init(),
  631. },
  632. },
  633. fields: []ast.Field{
  634. {
  635. Expr: &ast.FieldRef{Name: "id1", StreamName: "src1"},
  636. Name: "id1",
  637. AName: "",
  638. },
  639. },
  640. isAggregate: false,
  641. sendMeta: false,
  642. }.Init(),
  643. },
  644. { // 6. optimize outter join on
  645. sql: `SELECT id1 FROM src1 FULL JOIN src2 on src1.id1 = src2.id2 and src1.temp > 20 and src2.hum < 60 WHERE src1.id1 > 111 GROUP BY TUMBLINGWINDOW(ss, 10)`,
  646. p: ProjectPlan{
  647. baseLogicalPlan: baseLogicalPlan{
  648. children: []LogicalPlan{
  649. JoinPlan{
  650. baseLogicalPlan: baseLogicalPlan{
  651. children: []LogicalPlan{
  652. WindowPlan{
  653. baseLogicalPlan: baseLogicalPlan{
  654. children: []LogicalPlan{
  655. FilterPlan{
  656. baseLogicalPlan: baseLogicalPlan{
  657. children: []LogicalPlan{
  658. DataSourcePlan{
  659. name: "src1",
  660. streamFields: map[string]*ast.JsonStreamField{
  661. "id1": {
  662. Type: "bigint",
  663. },
  664. "temp": {
  665. Type: "bigint",
  666. },
  667. },
  668. streamStmt: streams["src1"],
  669. metaFields: []string{},
  670. }.Init(),
  671. },
  672. },
  673. condition: &ast.BinaryExpr{
  674. OP: ast.GT,
  675. LHS: &ast.FieldRef{Name: "id1", StreamName: "src1"},
  676. RHS: &ast.IntegerLiteral{Val: 111},
  677. },
  678. }.Init(),
  679. DataSourcePlan{
  680. name: "src2",
  681. streamFields: map[string]*ast.JsonStreamField{
  682. "hum": {
  683. Type: "bigint",
  684. },
  685. "id2": {
  686. Type: "bigint",
  687. },
  688. },
  689. streamStmt: streams["src2"],
  690. metaFields: []string{},
  691. timestampFormat: "YYYY-MM-dd HH:mm:ss",
  692. }.Init(),
  693. },
  694. },
  695. condition: nil,
  696. wtype: ast.TUMBLING_WINDOW,
  697. length: 10,
  698. timeUnit: ast.SS,
  699. interval: 0,
  700. limit: 0,
  701. }.Init(),
  702. },
  703. },
  704. from: &ast.Table{
  705. Name: "src1",
  706. },
  707. joins: []ast.Join{
  708. {
  709. Name: "src2",
  710. Alias: "",
  711. JoinType: ast.FULL_JOIN,
  712. Expr: &ast.BinaryExpr{
  713. OP: ast.AND,
  714. LHS: &ast.BinaryExpr{
  715. OP: ast.AND,
  716. LHS: &ast.BinaryExpr{
  717. LHS: &ast.FieldRef{Name: "id1", StreamName: "src1"},
  718. OP: ast.EQ,
  719. RHS: &ast.FieldRef{Name: "id2", StreamName: "src2"},
  720. },
  721. RHS: &ast.BinaryExpr{
  722. OP: ast.GT,
  723. LHS: &ast.FieldRef{Name: "temp", StreamName: "src1"},
  724. RHS: &ast.IntegerLiteral{Val: 20},
  725. },
  726. },
  727. RHS: &ast.BinaryExpr{
  728. OP: ast.LT,
  729. LHS: &ast.FieldRef{Name: "hum", StreamName: "src2"},
  730. RHS: &ast.IntegerLiteral{Val: 60},
  731. },
  732. },
  733. },
  734. },
  735. }.Init(),
  736. },
  737. },
  738. fields: []ast.Field{
  739. {
  740. Expr: &ast.FieldRef{Name: "id1", StreamName: "src1"},
  741. Name: "id1",
  742. AName: "",
  743. },
  744. },
  745. isAggregate: false,
  746. sendMeta: false,
  747. }.Init(),
  748. },
  749. { // 7 window error for table
  750. sql: `SELECT value FROM tableInPlanner WHERE name = "v1" GROUP BY TUMBLINGWINDOW(ss, 10)`,
  751. p: nil,
  752. err: "cannot run window for TABLE sources",
  753. },
  754. { // 8 join table without window
  755. sql: `SELECT id1 FROM src1 INNER JOIN tableInPlanner on src1.id1 = tableInPlanner.id and src1.temp > 20 and hum < 60 WHERE src1.id1 > 111`,
  756. p: ProjectPlan{
  757. baseLogicalPlan: baseLogicalPlan{
  758. children: []LogicalPlan{
  759. JoinPlan{
  760. baseLogicalPlan: baseLogicalPlan{
  761. children: []LogicalPlan{
  762. JoinAlignPlan{
  763. baseLogicalPlan: baseLogicalPlan{
  764. children: []LogicalPlan{
  765. FilterPlan{
  766. baseLogicalPlan: baseLogicalPlan{
  767. children: []LogicalPlan{
  768. DataSourcePlan{
  769. name: "src1",
  770. streamFields: map[string]*ast.JsonStreamField{
  771. "id1": {
  772. Type: "bigint",
  773. },
  774. "temp": {
  775. Type: "bigint",
  776. },
  777. },
  778. streamStmt: streams["src1"],
  779. metaFields: []string{},
  780. }.Init(),
  781. },
  782. },
  783. condition: &ast.BinaryExpr{
  784. RHS: &ast.BinaryExpr{
  785. OP: ast.GT,
  786. LHS: &ast.FieldRef{Name: "temp", StreamName: "src1"},
  787. RHS: &ast.IntegerLiteral{Val: 20},
  788. },
  789. OP: ast.AND,
  790. LHS: &ast.BinaryExpr{
  791. OP: ast.GT,
  792. LHS: &ast.FieldRef{Name: "id1", StreamName: "src1"},
  793. RHS: &ast.IntegerLiteral{Val: 111},
  794. },
  795. },
  796. }.Init(),
  797. DataSourcePlan{
  798. name: "tableInPlanner",
  799. streamFields: map[string]*ast.JsonStreamField{
  800. "hum": {
  801. Type: "bigint",
  802. },
  803. "id": {
  804. Type: "bigint",
  805. },
  806. },
  807. streamStmt: streams["tableInPlanner"],
  808. metaFields: []string{},
  809. }.Init(),
  810. },
  811. },
  812. Emitters: []string{"tableInPlanner"},
  813. }.Init(),
  814. },
  815. },
  816. from: &ast.Table{
  817. Name: "src1",
  818. },
  819. joins: []ast.Join{
  820. {
  821. Name: "tableInPlanner",
  822. Alias: "",
  823. JoinType: ast.INNER_JOIN,
  824. Expr: &ast.BinaryExpr{
  825. OP: ast.AND,
  826. LHS: &ast.BinaryExpr{
  827. LHS: &ast.FieldRef{Name: "id1", StreamName: "src1"},
  828. OP: ast.EQ,
  829. RHS: &ast.FieldRef{Name: "id", StreamName: "tableInPlanner"},
  830. },
  831. RHS: &ast.BinaryExpr{
  832. OP: ast.LT,
  833. LHS: &ast.FieldRef{Name: "hum", StreamName: "tableInPlanner"},
  834. RHS: &ast.IntegerLiteral{Val: 60},
  835. },
  836. },
  837. },
  838. },
  839. }.Init(),
  840. },
  841. },
  842. fields: []ast.Field{
  843. {
  844. Expr: &ast.FieldRef{Name: "id1", StreamName: "src1"},
  845. Name: "id1",
  846. AName: "",
  847. },
  848. },
  849. isAggregate: false,
  850. sendMeta: false,
  851. }.Init(),
  852. },
  853. { // 9 join table with window
  854. sql: `SELECT id1 FROM src1 INNER JOIN tableInPlanner on src1.id1 = tableInPlanner.id and src1.temp > 20 and tableInPlanner.hum < 60 WHERE src1.id1 > 111 GROUP BY TUMBLINGWINDOW(ss, 10)`,
  855. p: ProjectPlan{
  856. baseLogicalPlan: baseLogicalPlan{
  857. children: []LogicalPlan{
  858. JoinPlan{
  859. baseLogicalPlan: baseLogicalPlan{
  860. children: []LogicalPlan{
  861. JoinAlignPlan{
  862. baseLogicalPlan: baseLogicalPlan{
  863. children: []LogicalPlan{
  864. WindowPlan{
  865. baseLogicalPlan: baseLogicalPlan{
  866. children: []LogicalPlan{
  867. DataSourcePlan{
  868. name: "src1",
  869. streamFields: map[string]*ast.JsonStreamField{
  870. "id1": {
  871. Type: "bigint",
  872. },
  873. "temp": {
  874. Type: "bigint",
  875. },
  876. },
  877. streamStmt: streams["src1"],
  878. metaFields: []string{},
  879. }.Init(),
  880. },
  881. },
  882. condition: nil,
  883. wtype: ast.TUMBLING_WINDOW,
  884. length: 10,
  885. timeUnit: ast.SS,
  886. interval: 0,
  887. limit: 0,
  888. }.Init(),
  889. DataSourcePlan{
  890. name: "tableInPlanner",
  891. streamFields: map[string]*ast.JsonStreamField{
  892. "hum": {
  893. Type: "bigint",
  894. },
  895. "id": {
  896. Type: "bigint",
  897. },
  898. },
  899. streamStmt: streams["tableInPlanner"],
  900. metaFields: []string{},
  901. }.Init(),
  902. },
  903. },
  904. Emitters: []string{"tableInPlanner"},
  905. }.Init(),
  906. },
  907. },
  908. from: &ast.Table{
  909. Name: "src1",
  910. },
  911. joins: []ast.Join{
  912. {
  913. Name: "tableInPlanner",
  914. Alias: "",
  915. JoinType: ast.INNER_JOIN,
  916. Expr: &ast.BinaryExpr{
  917. OP: ast.AND,
  918. LHS: &ast.BinaryExpr{
  919. LHS: &ast.FieldRef{Name: "id1", StreamName: "src1"},
  920. OP: ast.EQ,
  921. RHS: &ast.FieldRef{Name: "id", StreamName: "tableInPlanner"},
  922. },
  923. RHS: &ast.BinaryExpr{
  924. RHS: &ast.BinaryExpr{
  925. OP: ast.AND,
  926. LHS: &ast.BinaryExpr{
  927. OP: ast.GT,
  928. LHS: &ast.FieldRef{Name: "temp", StreamName: "src1"},
  929. RHS: &ast.IntegerLiteral{Val: 20},
  930. },
  931. RHS: &ast.BinaryExpr{
  932. OP: ast.LT,
  933. LHS: &ast.FieldRef{Name: "hum", StreamName: "tableInPlanner"},
  934. RHS: &ast.IntegerLiteral{Val: 60},
  935. },
  936. },
  937. OP: ast.AND,
  938. LHS: &ast.BinaryExpr{
  939. OP: ast.GT,
  940. LHS: &ast.FieldRef{Name: "id1", StreamName: "src1"},
  941. RHS: &ast.IntegerLiteral{Val: 111},
  942. },
  943. },
  944. },
  945. },
  946. },
  947. }.Init(),
  948. },
  949. },
  950. fields: []ast.Field{
  951. {
  952. Expr: &ast.FieldRef{Name: "id1", StreamName: "src1"},
  953. Name: "id1",
  954. AName: "",
  955. },
  956. },
  957. isAggregate: false,
  958. sendMeta: false,
  959. }.Init(),
  960. },
  961. { // 10 meta
  962. sql: `SELECT temp, meta(id) AS eid,meta(Humidity->Device) AS hdevice FROM src1 WHERE meta(device)="demo2"`,
  963. p: ProjectPlan{
  964. baseLogicalPlan: baseLogicalPlan{
  965. children: []LogicalPlan{
  966. FilterPlan{
  967. baseLogicalPlan: baseLogicalPlan{
  968. children: []LogicalPlan{
  969. DataSourcePlan{
  970. name: "src1",
  971. streamFields: map[string]*ast.JsonStreamField{
  972. "temp": {
  973. Type: "bigint",
  974. },
  975. },
  976. streamStmt: streams["src1"],
  977. metaFields: []string{"Humidity", "device", "id"},
  978. }.Init(),
  979. },
  980. },
  981. condition: &ast.BinaryExpr{
  982. LHS: &ast.Call{
  983. Name: "meta",
  984. FuncId: 2,
  985. Args: []ast.Expr{&ast.MetaRef{
  986. Name: "device",
  987. StreamName: ast.DefaultStream,
  988. }},
  989. },
  990. OP: ast.EQ,
  991. RHS: &ast.StringLiteral{
  992. Val: "demo2",
  993. },
  994. },
  995. }.Init(),
  996. },
  997. },
  998. fields: []ast.Field{
  999. {
  1000. Expr: &ast.FieldRef{Name: "temp", StreamName: "src1"},
  1001. Name: "temp",
  1002. AName: "",
  1003. }, {
  1004. Expr: &ast.FieldRef{Name: "eid", StreamName: ast.AliasStream, AliasRef: ast.MockAliasRef(
  1005. &ast.Call{Name: "meta", FuncId: 0, Args: []ast.Expr{&ast.MetaRef{
  1006. Name: "id",
  1007. StreamName: ast.DefaultStream,
  1008. }}},
  1009. []ast.StreamName{},
  1010. nil,
  1011. )},
  1012. Name: "meta",
  1013. AName: "eid",
  1014. }, {
  1015. Expr: &ast.FieldRef{Name: "hdevice", StreamName: ast.AliasStream, AliasRef: ast.MockAliasRef(
  1016. &ast.Call{Name: "meta", FuncId: 1, Args: []ast.Expr{
  1017. &ast.BinaryExpr{
  1018. OP: ast.ARROW,
  1019. LHS: &ast.MetaRef{Name: "Humidity", StreamName: ast.DefaultStream},
  1020. RHS: &ast.JsonFieldRef{Name: "Device"},
  1021. },
  1022. }},
  1023. []ast.StreamName{},
  1024. nil,
  1025. )},
  1026. Name: "meta",
  1027. AName: "hdevice",
  1028. },
  1029. },
  1030. isAggregate: false,
  1031. sendMeta: false,
  1032. }.Init(),
  1033. },
  1034. { // 11 join with same name field and aliased
  1035. sql: `SELECT src2.hum AS hum1, tableInPlanner.hum AS hum2 FROM src2 INNER JOIN tableInPlanner on id2 = id WHERE hum1 > hum2`,
  1036. p: ProjectPlan{
  1037. baseLogicalPlan: baseLogicalPlan{
  1038. children: []LogicalPlan{
  1039. JoinPlan{
  1040. baseLogicalPlan: baseLogicalPlan{
  1041. children: []LogicalPlan{
  1042. JoinAlignPlan{
  1043. baseLogicalPlan: baseLogicalPlan{
  1044. children: []LogicalPlan{
  1045. DataSourcePlan{
  1046. name: "src2",
  1047. streamFields: map[string]*ast.JsonStreamField{
  1048. "hum": {
  1049. Type: "bigint",
  1050. },
  1051. "id2": {
  1052. Type: "bigint",
  1053. },
  1054. },
  1055. streamStmt: streams["src2"],
  1056. metaFields: []string{},
  1057. timestampFormat: "YYYY-MM-dd HH:mm:ss",
  1058. }.Init(),
  1059. DataSourcePlan{
  1060. name: "tableInPlanner",
  1061. streamFields: map[string]*ast.JsonStreamField{
  1062. "hum": {
  1063. Type: "bigint",
  1064. },
  1065. "id": {
  1066. Type: "bigint",
  1067. },
  1068. },
  1069. streamStmt: streams["tableInPlanner"],
  1070. metaFields: []string{},
  1071. }.Init(),
  1072. },
  1073. },
  1074. Emitters: []string{"tableInPlanner"},
  1075. }.Init(),
  1076. },
  1077. },
  1078. from: &ast.Table{
  1079. Name: "src2",
  1080. },
  1081. joins: []ast.Join{
  1082. {
  1083. Name: "tableInPlanner",
  1084. Alias: "",
  1085. JoinType: ast.INNER_JOIN,
  1086. Expr: &ast.BinaryExpr{
  1087. RHS: &ast.BinaryExpr{
  1088. OP: ast.EQ,
  1089. LHS: &ast.FieldRef{Name: "id2", StreamName: "src2"},
  1090. RHS: &ast.FieldRef{Name: "id", StreamName: "tableInPlanner"},
  1091. },
  1092. OP: ast.AND,
  1093. LHS: &ast.BinaryExpr{
  1094. OP: ast.GT,
  1095. LHS: &ast.FieldRef{Name: "hum1", StreamName: ast.AliasStream, AliasRef: ast.MockAliasRef(
  1096. &ast.FieldRef{
  1097. Name: "hum",
  1098. StreamName: "src2",
  1099. },
  1100. []ast.StreamName{"src2"},
  1101. &boolFalse,
  1102. )},
  1103. RHS: &ast.FieldRef{Name: "hum2", StreamName: ast.AliasStream, AliasRef: ast.MockAliasRef(
  1104. &ast.FieldRef{
  1105. Name: "hum",
  1106. StreamName: "tableInPlanner",
  1107. },
  1108. []ast.StreamName{"tableInPlanner"},
  1109. &boolFalse,
  1110. )},
  1111. },
  1112. },
  1113. },
  1114. },
  1115. }.Init(),
  1116. },
  1117. },
  1118. fields: []ast.Field{
  1119. {
  1120. Expr: &ast.FieldRef{Name: "hum1", StreamName: ast.AliasStream, AliasRef: ast.MockAliasRef(
  1121. &ast.FieldRef{
  1122. Name: "hum",
  1123. StreamName: "src2",
  1124. },
  1125. []ast.StreamName{"src2"},
  1126. &boolFalse,
  1127. )},
  1128. Name: "hum",
  1129. AName: "hum1",
  1130. }, {
  1131. Expr: &ast.FieldRef{Name: "hum2", StreamName: ast.AliasStream, AliasRef: ast.MockAliasRef(
  1132. &ast.FieldRef{
  1133. Name: "hum",
  1134. StreamName: "tableInPlanner",
  1135. },
  1136. []ast.StreamName{"tableInPlanner"},
  1137. &boolFalse,
  1138. )},
  1139. Name: "hum",
  1140. AName: "hum2",
  1141. },
  1142. },
  1143. isAggregate: false,
  1144. sendMeta: false,
  1145. }.Init(),
  1146. },
  1147. { // 12 meta with more fields
  1148. sql: `SELECT temp, meta(*) as m FROM src1 WHERE meta(device)="demo2"`,
  1149. p: ProjectPlan{
  1150. baseLogicalPlan: baseLogicalPlan{
  1151. children: []LogicalPlan{
  1152. FilterPlan{
  1153. baseLogicalPlan: baseLogicalPlan{
  1154. children: []LogicalPlan{
  1155. DataSourcePlan{
  1156. name: "src1",
  1157. streamFields: map[string]*ast.JsonStreamField{
  1158. "temp": {
  1159. Type: "bigint",
  1160. },
  1161. },
  1162. streamStmt: streams["src1"],
  1163. metaFields: []string{},
  1164. allMeta: true,
  1165. }.Init(),
  1166. },
  1167. },
  1168. condition: &ast.BinaryExpr{
  1169. LHS: &ast.Call{
  1170. Name: "meta",
  1171. FuncId: 1,
  1172. Args: []ast.Expr{&ast.MetaRef{
  1173. Name: "device",
  1174. StreamName: ast.DefaultStream,
  1175. }},
  1176. },
  1177. OP: ast.EQ,
  1178. RHS: &ast.StringLiteral{
  1179. Val: "demo2",
  1180. },
  1181. },
  1182. }.Init(),
  1183. },
  1184. },
  1185. fields: []ast.Field{
  1186. {
  1187. Expr: &ast.FieldRef{Name: "temp", StreamName: "src1"},
  1188. Name: "temp",
  1189. AName: "",
  1190. }, {
  1191. Expr: &ast.FieldRef{Name: "m", StreamName: ast.AliasStream, AliasRef: ast.MockAliasRef(
  1192. &ast.Call{Name: "meta", FuncId: 0, Args: []ast.Expr{&ast.MetaRef{
  1193. Name: "*",
  1194. StreamName: ast.DefaultStream,
  1195. }}},
  1196. []ast.StreamName{},
  1197. nil,
  1198. )},
  1199. Name: "meta",
  1200. AName: "m",
  1201. },
  1202. },
  1203. isAggregate: false,
  1204. sendMeta: false,
  1205. }.Init(),
  1206. },
  1207. { // 13 analytic function plan
  1208. sql: `SELECT latest(lag(name)), id1 FROM src1 WHERE lag(temp) > temp`,
  1209. p: ProjectPlan{
  1210. baseLogicalPlan: baseLogicalPlan{
  1211. children: []LogicalPlan{
  1212. FilterPlan{
  1213. baseLogicalPlan: baseLogicalPlan{
  1214. children: []LogicalPlan{
  1215. AnalyticFuncsPlan{
  1216. baseLogicalPlan: baseLogicalPlan{
  1217. children: []LogicalPlan{
  1218. DataSourcePlan{
  1219. name: "src1",
  1220. streamFields: map[string]*ast.JsonStreamField{
  1221. "id1": {
  1222. Type: "bigint",
  1223. },
  1224. "name": {
  1225. Type: "string",
  1226. },
  1227. "temp": {
  1228. Type: "bigint",
  1229. },
  1230. },
  1231. streamStmt: streams["src1"],
  1232. metaFields: []string{},
  1233. }.Init(),
  1234. },
  1235. },
  1236. funcs: []*ast.Call{
  1237. {
  1238. Name: "lag",
  1239. FuncId: 2,
  1240. CachedField: "$$a_lag_2",
  1241. Args: []ast.Expr{&ast.FieldRef{
  1242. Name: "temp",
  1243. StreamName: "src1",
  1244. }},
  1245. },
  1246. {
  1247. Name: "latest", FuncId: 1, CachedField: "$$a_latest_1", FuncType: ast.FuncTypeScalar, Args: []ast.Expr{&ast.Call{Name: "lag", FuncId: 0, Cached: true, CachedField: "$$a_lag_0", FuncType: ast.FuncTypeScalar, Args: []ast.Expr{&ast.FieldRef{Name: "name", StreamName: "src1"}}}},
  1248. },
  1249. {
  1250. Name: "lag", FuncId: 0, CachedField: "$$a_lag_0", FuncType: ast.FuncTypeScalar, Args: []ast.Expr{&ast.FieldRef{Name: "name", StreamName: "src1"}},
  1251. },
  1252. },
  1253. }.Init(),
  1254. },
  1255. },
  1256. condition: &ast.BinaryExpr{
  1257. LHS: &ast.Call{
  1258. Name: "lag",
  1259. FuncId: 2,
  1260. Args: []ast.Expr{&ast.FieldRef{
  1261. Name: "temp",
  1262. StreamName: "src1",
  1263. }},
  1264. CachedField: "$$a_lag_2",
  1265. Cached: true,
  1266. },
  1267. OP: ast.GT,
  1268. RHS: &ast.FieldRef{
  1269. Name: "temp",
  1270. StreamName: "src1",
  1271. },
  1272. },
  1273. }.Init(),
  1274. },
  1275. },
  1276. fields: []ast.Field{
  1277. {
  1278. Expr: &ast.Call{
  1279. Name: "latest",
  1280. FuncId: 1,
  1281. Args: []ast.Expr{&ast.Call{Name: "lag", FuncId: 0, Cached: true, CachedField: "$$a_lag_0", FuncType: ast.FuncTypeScalar, Args: []ast.Expr{&ast.FieldRef{Name: "name", StreamName: "src1"}}}},
  1282. CachedField: "$$a_latest_1",
  1283. Cached: true,
  1284. },
  1285. Name: "latest",
  1286. }, {
  1287. Expr: &ast.FieldRef{Name: "id1", StreamName: "src1"},
  1288. Name: "id1",
  1289. },
  1290. },
  1291. isAggregate: false,
  1292. sendMeta: false,
  1293. }.Init(),
  1294. },
  1295. { // 14
  1296. sql: `SELECT name, *, meta(device) FROM src1`,
  1297. p: ProjectPlan{
  1298. baseLogicalPlan: baseLogicalPlan{
  1299. children: []LogicalPlan{
  1300. DataSourcePlan{
  1301. baseLogicalPlan: baseLogicalPlan{},
  1302. name: "src1",
  1303. streamFields: map[string]*ast.JsonStreamField{
  1304. "id1": {
  1305. Type: "bigint",
  1306. },
  1307. "temp": {
  1308. Type: "bigint",
  1309. },
  1310. "name": {
  1311. Type: "string",
  1312. },
  1313. "myarray": {
  1314. Type: "array",
  1315. Items: &ast.JsonStreamField{
  1316. Type: "string",
  1317. },
  1318. },
  1319. },
  1320. streamStmt: streams["src1"],
  1321. metaFields: []string{"device"},
  1322. isWildCard: true,
  1323. }.Init(),
  1324. },
  1325. },
  1326. fields: []ast.Field{
  1327. {
  1328. Expr: &ast.FieldRef{Name: "name", StreamName: "src1"},
  1329. Name: "name",
  1330. AName: "",
  1331. },
  1332. {
  1333. Name: "*",
  1334. Expr: &ast.Wildcard{
  1335. Token: ast.ASTERISK,
  1336. },
  1337. },
  1338. {
  1339. Name: "meta",
  1340. Expr: &ast.Call{
  1341. Name: "meta",
  1342. Args: []ast.Expr{
  1343. &ast.MetaRef{
  1344. StreamName: ast.DefaultStream,
  1345. Name: "device",
  1346. },
  1347. },
  1348. },
  1349. },
  1350. },
  1351. isAggregate: false,
  1352. allWildcard: true,
  1353. sendMeta: false,
  1354. }.Init(),
  1355. },
  1356. { // 15 analytic function over partition plan
  1357. sql: `SELECT latest(lag(name)) OVER (PARTITION BY temp), id1 FROM src1 WHERE lag(temp) > temp`,
  1358. p: ProjectPlan{
  1359. baseLogicalPlan: baseLogicalPlan{
  1360. children: []LogicalPlan{
  1361. FilterPlan{
  1362. baseLogicalPlan: baseLogicalPlan{
  1363. children: []LogicalPlan{
  1364. AnalyticFuncsPlan{
  1365. baseLogicalPlan: baseLogicalPlan{
  1366. children: []LogicalPlan{
  1367. DataSourcePlan{
  1368. name: "src1",
  1369. streamFields: map[string]*ast.JsonStreamField{
  1370. "id1": {
  1371. Type: "bigint",
  1372. },
  1373. "name": {
  1374. Type: "string",
  1375. },
  1376. "temp": {
  1377. Type: "bigint",
  1378. },
  1379. },
  1380. streamStmt: streams["src1"],
  1381. metaFields: []string{},
  1382. }.Init(),
  1383. },
  1384. },
  1385. funcs: []*ast.Call{
  1386. {
  1387. Name: "lag",
  1388. FuncId: 2,
  1389. CachedField: "$$a_lag_2",
  1390. Args: []ast.Expr{&ast.FieldRef{
  1391. Name: "temp",
  1392. StreamName: "src1",
  1393. }},
  1394. },
  1395. {
  1396. Name: "latest", FuncId: 1, CachedField: "$$a_latest_1", FuncType: ast.FuncTypeScalar, Args: []ast.Expr{&ast.Call{Name: "lag", FuncId: 0, Cached: true, CachedField: "$$a_lag_0", FuncType: ast.FuncTypeScalar, Args: []ast.Expr{&ast.FieldRef{Name: "name", StreamName: "src1"}}}}, Partition: &ast.PartitionExpr{Exprs: []ast.Expr{&ast.FieldRef{Name: "temp", StreamName: "src1"}}},
  1397. },
  1398. {
  1399. Name: "lag", FuncId: 0, CachedField: "$$a_lag_0", FuncType: ast.FuncTypeScalar, Args: []ast.Expr{&ast.FieldRef{Name: "name", StreamName: "src1"}},
  1400. },
  1401. },
  1402. }.Init(),
  1403. },
  1404. },
  1405. condition: &ast.BinaryExpr{
  1406. LHS: &ast.Call{
  1407. Name: "lag",
  1408. FuncId: 2,
  1409. Args: []ast.Expr{&ast.FieldRef{
  1410. Name: "temp",
  1411. StreamName: "src1",
  1412. }},
  1413. CachedField: "$$a_lag_2",
  1414. Cached: true,
  1415. },
  1416. OP: ast.GT,
  1417. RHS: &ast.FieldRef{
  1418. Name: "temp",
  1419. StreamName: "src1",
  1420. },
  1421. },
  1422. }.Init(),
  1423. },
  1424. },
  1425. fields: []ast.Field{
  1426. {
  1427. Expr: &ast.Call{
  1428. Name: "latest",
  1429. FuncId: 1,
  1430. Args: []ast.Expr{&ast.Call{Name: "lag", FuncId: 0, Cached: true, CachedField: "$$a_lag_0", FuncType: ast.FuncTypeScalar, Args: []ast.Expr{&ast.FieldRef{Name: "name", StreamName: "src1"}}}},
  1431. CachedField: "$$a_latest_1",
  1432. Cached: true,
  1433. Partition: &ast.PartitionExpr{
  1434. Exprs: []ast.Expr{
  1435. &ast.FieldRef{Name: "temp", StreamName: "src1"},
  1436. },
  1437. },
  1438. },
  1439. Name: "latest",
  1440. }, {
  1441. Expr: &ast.FieldRef{Name: "id1", StreamName: "src1"},
  1442. Name: "id1",
  1443. },
  1444. },
  1445. isAggregate: false,
  1446. sendMeta: false,
  1447. }.Init(),
  1448. },
  1449. { // 16 analytic function over partition when plan
  1450. sql: `SELECT latest(lag(name)) OVER (PARTITION BY temp WHEN temp > 12), id1 FROM src1 WHERE lag(temp) > temp`,
  1451. p: ProjectPlan{
  1452. baseLogicalPlan: baseLogicalPlan{
  1453. children: []LogicalPlan{
  1454. FilterPlan{
  1455. baseLogicalPlan: baseLogicalPlan{
  1456. children: []LogicalPlan{
  1457. AnalyticFuncsPlan{
  1458. baseLogicalPlan: baseLogicalPlan{
  1459. children: []LogicalPlan{
  1460. DataSourcePlan{
  1461. name: "src1",
  1462. streamFields: map[string]*ast.JsonStreamField{
  1463. "id1": {
  1464. Type: "bigint",
  1465. },
  1466. "name": {
  1467. Type: "string",
  1468. },
  1469. "temp": {
  1470. Type: "bigint",
  1471. },
  1472. },
  1473. streamStmt: streams["src1"],
  1474. metaFields: []string{},
  1475. }.Init(),
  1476. },
  1477. },
  1478. funcs: []*ast.Call{
  1479. {
  1480. Name: "lag",
  1481. FuncId: 2,
  1482. CachedField: "$$a_lag_2",
  1483. Args: []ast.Expr{&ast.FieldRef{
  1484. Name: "temp",
  1485. StreamName: "src1",
  1486. }},
  1487. },
  1488. {
  1489. Name: "latest", FuncId: 1, CachedField: "$$a_latest_1", FuncType: ast.FuncTypeScalar, Args: []ast.Expr{&ast.Call{Name: "lag", FuncId: 0, Cached: true, CachedField: "$$a_lag_0", FuncType: ast.FuncTypeScalar, Args: []ast.Expr{&ast.FieldRef{Name: "name", StreamName: "src1"}}}}, Partition: &ast.PartitionExpr{Exprs: []ast.Expr{&ast.FieldRef{Name: "temp", StreamName: "src1"}}}, WhenExpr: &ast.BinaryExpr{LHS: &ast.FieldRef{Name: "temp", StreamName: "src1"}, OP: ast.GT, RHS: &ast.IntegerLiteral{Val: 12}},
  1490. },
  1491. {
  1492. Name: "lag", FuncId: 0, CachedField: "$$a_lag_0", FuncType: ast.FuncTypeScalar, Args: []ast.Expr{&ast.FieldRef{Name: "name", StreamName: "src1"}},
  1493. },
  1494. },
  1495. }.Init(),
  1496. },
  1497. },
  1498. condition: &ast.BinaryExpr{
  1499. LHS: &ast.Call{
  1500. Name: "lag",
  1501. FuncId: 2,
  1502. Args: []ast.Expr{&ast.FieldRef{
  1503. Name: "temp",
  1504. StreamName: "src1",
  1505. }},
  1506. CachedField: "$$a_lag_2",
  1507. Cached: true,
  1508. },
  1509. OP: ast.GT,
  1510. RHS: &ast.FieldRef{
  1511. Name: "temp",
  1512. StreamName: "src1",
  1513. },
  1514. },
  1515. }.Init(),
  1516. },
  1517. },
  1518. fields: []ast.Field{
  1519. {
  1520. Expr: &ast.Call{
  1521. Name: "latest",
  1522. FuncId: 1,
  1523. Args: []ast.Expr{&ast.Call{Name: "lag", FuncId: 0, Cached: true, CachedField: "$$a_lag_0", FuncType: ast.FuncTypeScalar, Args: []ast.Expr{&ast.FieldRef{Name: "name", StreamName: "src1"}}}},
  1524. CachedField: "$$a_latest_1",
  1525. Cached: true,
  1526. Partition: &ast.PartitionExpr{
  1527. Exprs: []ast.Expr{
  1528. &ast.FieldRef{Name: "temp", StreamName: "src1"},
  1529. },
  1530. },
  1531. WhenExpr: &ast.BinaryExpr{
  1532. LHS: &ast.FieldRef{Name: "temp", StreamName: "src1"},
  1533. OP: ast.GT,
  1534. RHS: &ast.IntegerLiteral{Val: 12},
  1535. },
  1536. },
  1537. Name: "latest",
  1538. }, {
  1539. Expr: &ast.FieldRef{Name: "id1", StreamName: "src1"},
  1540. Name: "id1",
  1541. },
  1542. },
  1543. isAggregate: false,
  1544. sendMeta: false,
  1545. }.Init(),
  1546. },
  1547. { // 17. do not optimize sliding window
  1548. sql: `SELECT * FROM src1 WHERE temp > 20 GROUP BY SLIDINGWINDOW(ss, 10) HAVING COUNT(*) > 2`,
  1549. p: ProjectPlan{
  1550. baseLogicalPlan: baseLogicalPlan{
  1551. children: []LogicalPlan{
  1552. HavingPlan{
  1553. baseLogicalPlan: baseLogicalPlan{
  1554. children: []LogicalPlan{
  1555. FilterPlan{
  1556. baseLogicalPlan: baseLogicalPlan{
  1557. children: []LogicalPlan{
  1558. WindowPlan{
  1559. baseLogicalPlan: baseLogicalPlan{
  1560. children: []LogicalPlan{
  1561. DataSourcePlan{
  1562. name: "src1",
  1563. isWildCard: true,
  1564. streamFields: map[string]*ast.JsonStreamField{
  1565. "id1": {
  1566. Type: "bigint",
  1567. },
  1568. "temp": {
  1569. Type: "bigint",
  1570. },
  1571. "name": {
  1572. Type: "string",
  1573. },
  1574. "myarray": {
  1575. Type: "array",
  1576. Items: &ast.JsonStreamField{
  1577. Type: "string",
  1578. },
  1579. },
  1580. },
  1581. streamStmt: streams["src1"],
  1582. metaFields: []string{},
  1583. }.Init(),
  1584. },
  1585. },
  1586. condition: nil,
  1587. wtype: ast.SLIDING_WINDOW,
  1588. length: 10,
  1589. timeUnit: ast.SS,
  1590. interval: 0,
  1591. limit: 0,
  1592. }.Init(),
  1593. },
  1594. },
  1595. condition: &ast.BinaryExpr{
  1596. LHS: &ast.FieldRef{Name: "temp", StreamName: "src1"},
  1597. OP: ast.GT,
  1598. RHS: &ast.IntegerLiteral{Val: 20},
  1599. },
  1600. }.Init(),
  1601. },
  1602. },
  1603. condition: &ast.BinaryExpr{
  1604. LHS: &ast.Call{Name: "count", FuncId: 0, Args: []ast.Expr{&ast.Wildcard{
  1605. Token: ast.ASTERISK,
  1606. }}, FuncType: ast.FuncTypeAgg},
  1607. OP: ast.GT,
  1608. RHS: &ast.IntegerLiteral{Val: 2},
  1609. },
  1610. }.Init(),
  1611. },
  1612. },
  1613. fields: []ast.Field{
  1614. {
  1615. Expr: &ast.Wildcard{Token: ast.ASTERISK},
  1616. Name: "*",
  1617. AName: "",
  1618. },
  1619. },
  1620. isAggregate: false,
  1621. sendMeta: false,
  1622. }.Init(),
  1623. },
  1624. {
  1625. // 18 analytic function over when plan
  1626. sql: `SELECT CASE WHEN lag(temp) OVER (WHEN lag(id1) > 1) BETWEEN 0 AND 10 THEN 1 ELSE 0 END FROM src1`,
  1627. p: ProjectPlan{
  1628. baseLogicalPlan: baseLogicalPlan{
  1629. children: []LogicalPlan{
  1630. AnalyticFuncsPlan{
  1631. baseLogicalPlan: baseLogicalPlan{
  1632. children: []LogicalPlan{
  1633. DataSourcePlan{
  1634. name: "src1",
  1635. streamFields: map[string]*ast.JsonStreamField{
  1636. "id1": {
  1637. Type: "bigint",
  1638. },
  1639. "temp": {
  1640. Type: "bigint",
  1641. },
  1642. },
  1643. streamStmt: &ast.StreamStmt{
  1644. Name: "src1",
  1645. StreamFields: []ast.StreamField{
  1646. {
  1647. Name: "id1",
  1648. FieldType: &ast.BasicType{
  1649. Type: ast.DataType(1),
  1650. },
  1651. },
  1652. {
  1653. Name: "temp",
  1654. FieldType: &ast.BasicType{
  1655. Type: ast.DataType(1),
  1656. },
  1657. },
  1658. {
  1659. Name: "name",
  1660. FieldType: &ast.BasicType{
  1661. Type: ast.DataType(3),
  1662. },
  1663. },
  1664. {
  1665. Name: "myarray",
  1666. FieldType: &ast.ArrayType{
  1667. Type: ast.DataType(3),
  1668. },
  1669. },
  1670. },
  1671. Options: &ast.Options{
  1672. DATASOURCE: "src1",
  1673. KEY: "ts",
  1674. FORMAT: "json",
  1675. },
  1676. StreamType: ast.StreamType(0),
  1677. },
  1678. metaFields: []string{},
  1679. }.Init(),
  1680. },
  1681. },
  1682. funcs: []*ast.Call{
  1683. {
  1684. Name: "lag",
  1685. FuncId: 0,
  1686. FuncType: ast.FuncType(0),
  1687. Args: []ast.Expr{
  1688. &ast.FieldRef{
  1689. StreamName: "src1",
  1690. Name: "temp",
  1691. },
  1692. },
  1693. CachedField: "$$a_lag_0",
  1694. WhenExpr: &ast.BinaryExpr{
  1695. OP: ast.GT,
  1696. LHS: &ast.Call{
  1697. Name: "lag",
  1698. FuncId: 1,
  1699. FuncType: ast.FuncType(0),
  1700. Args: []ast.Expr{
  1701. &ast.FieldRef{
  1702. StreamName: "src1",
  1703. Name: "id1",
  1704. },
  1705. },
  1706. CachedField: "$$a_lag_1",
  1707. Cached: true,
  1708. },
  1709. RHS: &ast.IntegerLiteral{
  1710. Val: 1,
  1711. },
  1712. },
  1713. },
  1714. {
  1715. Name: "lag",
  1716. FuncId: 1,
  1717. FuncType: ast.FuncType(0),
  1718. Args: []ast.Expr{
  1719. &ast.FieldRef{
  1720. StreamName: "src1",
  1721. Name: "id1",
  1722. },
  1723. },
  1724. CachedField: "$$a_lag_1",
  1725. },
  1726. },
  1727. }.Init(),
  1728. },
  1729. },
  1730. fields: []ast.Field{
  1731. {
  1732. Name: "kuiper_field_0",
  1733. Expr: &ast.CaseExpr{
  1734. WhenClauses: []*ast.WhenClause{
  1735. {
  1736. Expr: &ast.BinaryExpr{
  1737. OP: ast.BETWEEN,
  1738. LHS: &ast.Call{
  1739. Name: "lag",
  1740. FuncId: 0,
  1741. FuncType: ast.FuncType(0),
  1742. Args: []ast.Expr{
  1743. &ast.FieldRef{
  1744. StreamName: "src1",
  1745. Name: "temp",
  1746. },
  1747. },
  1748. CachedField: "$$a_lag_0",
  1749. Cached: true,
  1750. WhenExpr: &ast.BinaryExpr{
  1751. OP: ast.GT,
  1752. LHS: &ast.Call{
  1753. Name: "lag",
  1754. FuncId: 1,
  1755. FuncType: ast.FuncType(0),
  1756. Args: []ast.Expr{
  1757. &ast.FieldRef{
  1758. StreamName: "src1",
  1759. Name: "id1",
  1760. },
  1761. },
  1762. CachedField: "$$a_lag_1",
  1763. Cached: true,
  1764. },
  1765. RHS: &ast.IntegerLiteral{
  1766. Val: 1,
  1767. },
  1768. },
  1769. },
  1770. RHS: &ast.BetweenExpr{
  1771. Lower: &ast.IntegerLiteral{
  1772. Val: 0,
  1773. },
  1774. Higher: &ast.IntegerLiteral{
  1775. Val: 10,
  1776. },
  1777. },
  1778. },
  1779. Result: &ast.IntegerLiteral{
  1780. Val: 1,
  1781. },
  1782. },
  1783. },
  1784. ElseClause: &ast.IntegerLiteral{
  1785. Val: 0,
  1786. },
  1787. },
  1788. },
  1789. },
  1790. }.Init(),
  1791. },
  1792. }
  1793. fmt.Printf("The test bucket size is %d.\n\n", len(tests))
  1794. for i, tt := range tests {
  1795. stmt, err := xsql.NewParser(strings.NewReader(tt.sql)).Parse()
  1796. if err != nil {
  1797. t.Errorf("%d. %q: error compile sql: %s\n", i, tt.sql, err)
  1798. continue
  1799. }
  1800. p, err := createLogicalPlan(stmt, &api.RuleOption{
  1801. IsEventTime: false,
  1802. LateTol: 0,
  1803. Concurrency: 0,
  1804. BufferLength: 0,
  1805. SendMetaToSink: false,
  1806. Qos: 0,
  1807. CheckpointInterval: 0,
  1808. SendError: true,
  1809. }, kv)
  1810. if !reflect.DeepEqual(tt.err, testx.Errstring(err)) {
  1811. t.Errorf("%d. %q: error mismatch:\n exp=%s\n got=%s\n\n", i, tt.sql, tt.err, err)
  1812. } else if !reflect.DeepEqual(tt.p, p) {
  1813. t.Errorf("%d. %q\n\nstmt mismatch:\n\nexp=%#v\n\ngot=%#v\n\n", i, tt.sql, render.AsCode(tt.p), render.AsCode(p))
  1814. }
  1815. }
  1816. }
  1817. func Test_createLogicalPlanSchemaless(t *testing.T) {
  1818. kv, err := store.GetKV("stream")
  1819. if err != nil {
  1820. t.Error(err)
  1821. return
  1822. }
  1823. streamSqls := map[string]string{
  1824. "src1": `CREATE STREAM src1 (
  1825. ) WITH (DATASOURCE="src1", FORMAT="json", KEY="ts");`,
  1826. "src2": `CREATE STREAM src2 (
  1827. ) WITH (DATASOURCE="src2", FORMAT="json", KEY="ts");`,
  1828. "tableInPlanner": `CREATE TABLE tableInPlanner (
  1829. id BIGINT,
  1830. name STRING,
  1831. value STRING,
  1832. hum BIGINT
  1833. ) WITH (TYPE="file");`,
  1834. }
  1835. types := map[string]ast.StreamType{
  1836. "src1": ast.TypeStream,
  1837. "src2": ast.TypeStream,
  1838. "tableInPlanner": ast.TypeTable,
  1839. }
  1840. for name, sql := range streamSqls {
  1841. s, err := json.Marshal(&xsql.StreamInfo{
  1842. StreamType: types[name],
  1843. Statement: sql,
  1844. })
  1845. if err != nil {
  1846. t.Error(err)
  1847. t.Fail()
  1848. }
  1849. err = kv.Set(name, string(s))
  1850. if err != nil {
  1851. t.Error(err)
  1852. t.Fail()
  1853. }
  1854. }
  1855. streams := make(map[string]*ast.StreamStmt)
  1856. for n := range streamSqls {
  1857. streamStmt, err := xsql.GetDataSource(kv, n)
  1858. if err != nil {
  1859. t.Errorf("fail to get stream %s, please check if stream is created", n)
  1860. return
  1861. }
  1862. streams[n] = streamStmt
  1863. }
  1864. // boolTrue = true
  1865. boolFalse := false
  1866. tests := []struct {
  1867. sql string
  1868. p LogicalPlan
  1869. err string
  1870. }{
  1871. { // 0
  1872. sql: `SELECT name FROM src1`,
  1873. p: ProjectPlan{
  1874. baseLogicalPlan: baseLogicalPlan{
  1875. children: []LogicalPlan{
  1876. DataSourcePlan{
  1877. baseLogicalPlan: baseLogicalPlan{},
  1878. name: "src1",
  1879. streamFields: map[string]*ast.JsonStreamField{
  1880. "name": nil,
  1881. },
  1882. streamStmt: streams["src1"],
  1883. isSchemaless: true,
  1884. metaFields: []string{},
  1885. }.Init(),
  1886. },
  1887. },
  1888. fields: []ast.Field{
  1889. {
  1890. Expr: &ast.FieldRef{Name: "name", StreamName: "src1"},
  1891. Name: "name",
  1892. AName: "",
  1893. },
  1894. },
  1895. isAggregate: false,
  1896. sendMeta: false,
  1897. }.Init(),
  1898. }, { // 1 optimize where to data source
  1899. sql: `SELECT temp FROM src1 WHERE name = "v1" GROUP BY TUMBLINGWINDOW(ss, 10)`,
  1900. p: ProjectPlan{
  1901. baseLogicalPlan: baseLogicalPlan{
  1902. children: []LogicalPlan{
  1903. WindowPlan{
  1904. baseLogicalPlan: baseLogicalPlan{
  1905. children: []LogicalPlan{
  1906. FilterPlan{
  1907. baseLogicalPlan: baseLogicalPlan{
  1908. children: []LogicalPlan{
  1909. DataSourcePlan{
  1910. name: "src1",
  1911. streamFields: map[string]*ast.JsonStreamField{
  1912. "name": nil,
  1913. "temp": nil,
  1914. },
  1915. streamStmt: streams["src1"],
  1916. metaFields: []string{},
  1917. isSchemaless: true,
  1918. }.Init(),
  1919. },
  1920. },
  1921. condition: &ast.BinaryExpr{
  1922. LHS: &ast.FieldRef{Name: "name", StreamName: "src1"},
  1923. OP: ast.EQ,
  1924. RHS: &ast.StringLiteral{Val: "v1"},
  1925. },
  1926. }.Init(),
  1927. },
  1928. },
  1929. condition: nil,
  1930. wtype: ast.TUMBLING_WINDOW,
  1931. length: 10,
  1932. timeUnit: ast.SS,
  1933. interval: 0,
  1934. limit: 0,
  1935. }.Init(),
  1936. },
  1937. },
  1938. fields: []ast.Field{
  1939. {
  1940. Expr: &ast.FieldRef{Name: "temp", StreamName: "src1"},
  1941. Name: "temp",
  1942. AName: "",
  1943. },
  1944. },
  1945. isAggregate: false,
  1946. sendMeta: false,
  1947. }.Init(),
  1948. }, { // 2 condition that cannot be optimized
  1949. sql: `SELECT id1 FROM src1 INNER JOIN src2 on src1.id1 = src2.id2 WHERE src1.temp > 20 OR src2.hum > 60 GROUP BY TUMBLINGWINDOW(ss, 10)`,
  1950. p: ProjectPlan{
  1951. baseLogicalPlan: baseLogicalPlan{
  1952. children: []LogicalPlan{
  1953. JoinPlan{
  1954. baseLogicalPlan: baseLogicalPlan{
  1955. children: []LogicalPlan{
  1956. WindowPlan{
  1957. baseLogicalPlan: baseLogicalPlan{
  1958. children: []LogicalPlan{
  1959. DataSourcePlan{
  1960. name: "src1",
  1961. streamFields: map[string]*ast.JsonStreamField{
  1962. "id1": nil,
  1963. "temp": nil,
  1964. },
  1965. streamStmt: streams["src1"],
  1966. metaFields: []string{},
  1967. isSchemaless: true,
  1968. }.Init(),
  1969. DataSourcePlan{
  1970. name: "src2",
  1971. streamFields: map[string]*ast.JsonStreamField{ // can't determine where is id1 belonged to
  1972. "hum": nil,
  1973. "id1": nil,
  1974. "id2": nil,
  1975. },
  1976. isSchemaless: true,
  1977. streamStmt: streams["src2"],
  1978. metaFields: []string{},
  1979. }.Init(),
  1980. },
  1981. },
  1982. condition: nil,
  1983. wtype: ast.TUMBLING_WINDOW,
  1984. length: 10,
  1985. timeUnit: ast.SS,
  1986. interval: 0,
  1987. limit: 0,
  1988. }.Init(),
  1989. },
  1990. },
  1991. from: &ast.Table{Name: "src1"},
  1992. joins: ast.Joins{ast.Join{
  1993. Name: "src2",
  1994. JoinType: ast.INNER_JOIN,
  1995. Expr: &ast.BinaryExpr{
  1996. OP: ast.AND,
  1997. LHS: &ast.BinaryExpr{
  1998. LHS: &ast.BinaryExpr{
  1999. OP: ast.GT,
  2000. LHS: &ast.FieldRef{Name: "temp", StreamName: "src1"},
  2001. RHS: &ast.IntegerLiteral{Val: 20},
  2002. },
  2003. OP: ast.OR,
  2004. RHS: &ast.BinaryExpr{
  2005. OP: ast.GT,
  2006. LHS: &ast.FieldRef{Name: "hum", StreamName: "src2"},
  2007. RHS: &ast.IntegerLiteral{Val: 60},
  2008. },
  2009. },
  2010. RHS: &ast.BinaryExpr{
  2011. OP: ast.EQ,
  2012. LHS: &ast.FieldRef{Name: "id1", StreamName: "src1"},
  2013. RHS: &ast.FieldRef{Name: "id2", StreamName: "src2"},
  2014. },
  2015. },
  2016. }},
  2017. }.Init(),
  2018. },
  2019. },
  2020. fields: []ast.Field{
  2021. {
  2022. Expr: &ast.FieldRef{Name: "id1", StreamName: ast.DefaultStream},
  2023. Name: "id1",
  2024. AName: "",
  2025. },
  2026. },
  2027. isAggregate: false,
  2028. sendMeta: false,
  2029. }.Init(),
  2030. }, { // 3 optimize window filter
  2031. sql: `SELECT id1 FROM src1 WHERE name = "v1" GROUP BY TUMBLINGWINDOW(ss, 10) FILTER( WHERE temp > 2)`,
  2032. p: ProjectPlan{
  2033. baseLogicalPlan: baseLogicalPlan{
  2034. children: []LogicalPlan{
  2035. WindowPlan{
  2036. baseLogicalPlan: baseLogicalPlan{
  2037. children: []LogicalPlan{
  2038. FilterPlan{
  2039. baseLogicalPlan: baseLogicalPlan{
  2040. children: []LogicalPlan{
  2041. DataSourcePlan{
  2042. name: "src1",
  2043. streamFields: map[string]*ast.JsonStreamField{
  2044. "id1": nil,
  2045. "name": nil,
  2046. "temp": nil,
  2047. },
  2048. isSchemaless: true,
  2049. streamStmt: streams["src1"],
  2050. metaFields: []string{},
  2051. }.Init(),
  2052. },
  2053. },
  2054. condition: &ast.BinaryExpr{
  2055. OP: ast.AND,
  2056. LHS: &ast.BinaryExpr{
  2057. LHS: &ast.FieldRef{Name: "name", StreamName: "src1"},
  2058. OP: ast.EQ,
  2059. RHS: &ast.StringLiteral{Val: "v1"},
  2060. },
  2061. RHS: &ast.BinaryExpr{
  2062. LHS: &ast.FieldRef{Name: "temp", StreamName: "src1"},
  2063. OP: ast.GT,
  2064. RHS: &ast.IntegerLiteral{Val: 2},
  2065. },
  2066. },
  2067. }.Init(),
  2068. },
  2069. },
  2070. condition: nil,
  2071. wtype: ast.TUMBLING_WINDOW,
  2072. length: 10,
  2073. timeUnit: ast.SS,
  2074. interval: 0,
  2075. limit: 0,
  2076. }.Init(),
  2077. },
  2078. },
  2079. fields: []ast.Field{
  2080. {
  2081. Expr: &ast.FieldRef{Name: "id1", StreamName: "src1"},
  2082. Name: "id1",
  2083. AName: "",
  2084. },
  2085. },
  2086. isAggregate: false,
  2087. sendMeta: false,
  2088. }.Init(),
  2089. }, { // 4. do not optimize count window
  2090. sql: `SELECT * FROM src1 WHERE temp > 20 GROUP BY COUNTWINDOW(5,1) HAVING COUNT(*) > 2`,
  2091. p: ProjectPlan{
  2092. baseLogicalPlan: baseLogicalPlan{
  2093. children: []LogicalPlan{
  2094. HavingPlan{
  2095. baseLogicalPlan: baseLogicalPlan{
  2096. children: []LogicalPlan{
  2097. FilterPlan{
  2098. baseLogicalPlan: baseLogicalPlan{
  2099. children: []LogicalPlan{
  2100. WindowPlan{
  2101. baseLogicalPlan: baseLogicalPlan{
  2102. children: []LogicalPlan{
  2103. DataSourcePlan{
  2104. name: "src1",
  2105. isWildCard: true,
  2106. streamFields: map[string]*ast.JsonStreamField{},
  2107. streamStmt: streams["src1"],
  2108. metaFields: []string{},
  2109. isSchemaless: true,
  2110. }.Init(),
  2111. },
  2112. },
  2113. condition: nil,
  2114. wtype: ast.COUNT_WINDOW,
  2115. length: 5,
  2116. interval: 1,
  2117. limit: 0,
  2118. }.Init(),
  2119. },
  2120. },
  2121. condition: &ast.BinaryExpr{
  2122. LHS: &ast.FieldRef{Name: "temp", StreamName: "src1"},
  2123. OP: ast.GT,
  2124. RHS: &ast.IntegerLiteral{Val: 20},
  2125. },
  2126. }.Init(),
  2127. },
  2128. },
  2129. condition: &ast.BinaryExpr{
  2130. LHS: &ast.Call{Name: "count", FuncId: 0, Args: []ast.Expr{&ast.Wildcard{
  2131. Token: ast.ASTERISK,
  2132. }}, FuncType: ast.FuncTypeAgg},
  2133. OP: ast.GT,
  2134. RHS: &ast.IntegerLiteral{Val: 2},
  2135. },
  2136. }.Init(),
  2137. },
  2138. },
  2139. fields: []ast.Field{
  2140. {
  2141. Expr: &ast.Wildcard{Token: ast.ASTERISK},
  2142. Name: "*",
  2143. AName: "",
  2144. },
  2145. },
  2146. isAggregate: false,
  2147. sendMeta: false,
  2148. }.Init(),
  2149. }, { // 5. optimize join on
  2150. sql: `SELECT id1 FROM src1 INNER JOIN src2 on src1.id1 = src2.id2 and src1.temp > 20 and src2.hum < 60 WHERE src1.id1 > 111 GROUP BY TUMBLINGWINDOW(ss, 10)`,
  2151. p: ProjectPlan{
  2152. baseLogicalPlan: baseLogicalPlan{
  2153. children: []LogicalPlan{
  2154. JoinPlan{
  2155. baseLogicalPlan: baseLogicalPlan{
  2156. children: []LogicalPlan{
  2157. WindowPlan{
  2158. baseLogicalPlan: baseLogicalPlan{
  2159. children: []LogicalPlan{
  2160. FilterPlan{
  2161. baseLogicalPlan: baseLogicalPlan{
  2162. children: []LogicalPlan{
  2163. DataSourcePlan{
  2164. name: "src1",
  2165. streamFields: map[string]*ast.JsonStreamField{
  2166. "id1": nil,
  2167. "temp": nil,
  2168. },
  2169. isSchemaless: true,
  2170. streamStmt: streams["src1"],
  2171. metaFields: []string{},
  2172. }.Init(),
  2173. },
  2174. },
  2175. condition: &ast.BinaryExpr{
  2176. RHS: &ast.BinaryExpr{
  2177. OP: ast.GT,
  2178. LHS: &ast.FieldRef{Name: "temp", StreamName: "src1"},
  2179. RHS: &ast.IntegerLiteral{Val: 20},
  2180. },
  2181. OP: ast.AND,
  2182. LHS: &ast.BinaryExpr{
  2183. OP: ast.GT,
  2184. LHS: &ast.FieldRef{Name: "id1", StreamName: "src1"},
  2185. RHS: &ast.IntegerLiteral{Val: 111},
  2186. },
  2187. },
  2188. }.Init(),
  2189. FilterPlan{
  2190. baseLogicalPlan: baseLogicalPlan{
  2191. children: []LogicalPlan{
  2192. DataSourcePlan{
  2193. name: "src2",
  2194. streamFields: map[string]*ast.JsonStreamField{
  2195. "hum": nil,
  2196. "id1": nil,
  2197. "id2": nil,
  2198. },
  2199. isSchemaless: true,
  2200. streamStmt: streams["src2"],
  2201. metaFields: []string{},
  2202. }.Init(),
  2203. },
  2204. },
  2205. condition: &ast.BinaryExpr{
  2206. OP: ast.LT,
  2207. LHS: &ast.FieldRef{Name: "hum", StreamName: "src2"},
  2208. RHS: &ast.IntegerLiteral{Val: 60},
  2209. },
  2210. }.Init(),
  2211. },
  2212. },
  2213. condition: nil,
  2214. wtype: ast.TUMBLING_WINDOW,
  2215. length: 10,
  2216. timeUnit: ast.SS,
  2217. interval: 0,
  2218. limit: 0,
  2219. }.Init(),
  2220. },
  2221. },
  2222. from: &ast.Table{
  2223. Name: "src1",
  2224. },
  2225. joins: []ast.Join{
  2226. {
  2227. Name: "src2",
  2228. Alias: "",
  2229. JoinType: ast.INNER_JOIN,
  2230. Expr: &ast.BinaryExpr{
  2231. LHS: &ast.FieldRef{Name: "id1", StreamName: "src1"},
  2232. OP: ast.EQ,
  2233. RHS: &ast.FieldRef{Name: "id2", StreamName: "src2"},
  2234. },
  2235. },
  2236. },
  2237. }.Init(),
  2238. },
  2239. },
  2240. fields: []ast.Field{
  2241. {
  2242. Expr: &ast.FieldRef{Name: "id1", StreamName: ast.DefaultStream},
  2243. Name: "id1",
  2244. AName: "",
  2245. },
  2246. },
  2247. isAggregate: false,
  2248. sendMeta: false,
  2249. }.Init(),
  2250. }, { // 6. optimize outter join on
  2251. sql: `SELECT id1 FROM src1 FULL JOIN src2 on src1.id1 = src2.id2 and src1.temp > 20 and src2.hum < 60 WHERE src1.id1 > 111 GROUP BY TUMBLINGWINDOW(ss, 10)`,
  2252. p: ProjectPlan{
  2253. baseLogicalPlan: baseLogicalPlan{
  2254. children: []LogicalPlan{
  2255. JoinPlan{
  2256. baseLogicalPlan: baseLogicalPlan{
  2257. children: []LogicalPlan{
  2258. WindowPlan{
  2259. baseLogicalPlan: baseLogicalPlan{
  2260. children: []LogicalPlan{
  2261. FilterPlan{
  2262. baseLogicalPlan: baseLogicalPlan{
  2263. children: []LogicalPlan{
  2264. DataSourcePlan{
  2265. name: "src1",
  2266. streamFields: map[string]*ast.JsonStreamField{
  2267. "id1": nil,
  2268. "temp": nil,
  2269. },
  2270. isSchemaless: true,
  2271. streamStmt: streams["src1"],
  2272. metaFields: []string{},
  2273. }.Init(),
  2274. },
  2275. },
  2276. condition: &ast.BinaryExpr{
  2277. OP: ast.GT,
  2278. LHS: &ast.FieldRef{Name: "id1", StreamName: "src1"},
  2279. RHS: &ast.IntegerLiteral{Val: 111},
  2280. },
  2281. }.Init(),
  2282. DataSourcePlan{
  2283. name: "src2",
  2284. streamFields: map[string]*ast.JsonStreamField{
  2285. "hum": nil,
  2286. "id1": nil,
  2287. "id2": nil,
  2288. },
  2289. isSchemaless: true,
  2290. streamStmt: streams["src2"],
  2291. metaFields: []string{},
  2292. }.Init(),
  2293. },
  2294. },
  2295. condition: nil,
  2296. wtype: ast.TUMBLING_WINDOW,
  2297. length: 10,
  2298. timeUnit: ast.SS,
  2299. interval: 0,
  2300. limit: 0,
  2301. }.Init(),
  2302. },
  2303. },
  2304. from: &ast.Table{
  2305. Name: "src1",
  2306. },
  2307. joins: []ast.Join{
  2308. {
  2309. Name: "src2",
  2310. Alias: "",
  2311. JoinType: ast.FULL_JOIN,
  2312. Expr: &ast.BinaryExpr{
  2313. OP: ast.AND,
  2314. LHS: &ast.BinaryExpr{
  2315. OP: ast.AND,
  2316. LHS: &ast.BinaryExpr{
  2317. LHS: &ast.FieldRef{Name: "id1", StreamName: "src1"},
  2318. OP: ast.EQ,
  2319. RHS: &ast.FieldRef{Name: "id2", StreamName: "src2"},
  2320. },
  2321. RHS: &ast.BinaryExpr{
  2322. OP: ast.GT,
  2323. LHS: &ast.FieldRef{Name: "temp", StreamName: "src1"},
  2324. RHS: &ast.IntegerLiteral{Val: 20},
  2325. },
  2326. },
  2327. RHS: &ast.BinaryExpr{
  2328. OP: ast.LT,
  2329. LHS: &ast.FieldRef{Name: "hum", StreamName: "src2"},
  2330. RHS: &ast.IntegerLiteral{Val: 60},
  2331. },
  2332. },
  2333. },
  2334. },
  2335. }.Init(),
  2336. },
  2337. },
  2338. fields: []ast.Field{
  2339. {
  2340. Expr: &ast.FieldRef{Name: "id1", StreamName: ast.DefaultStream},
  2341. Name: "id1",
  2342. AName: "",
  2343. },
  2344. },
  2345. isAggregate: false,
  2346. sendMeta: false,
  2347. }.Init(),
  2348. }, { // 7 window error for table
  2349. sql: `SELECT value FROM tableInPlanner WHERE name = "v1" GROUP BY TUMBLINGWINDOW(ss, 10)`,
  2350. p: nil,
  2351. err: "cannot run window for TABLE sources",
  2352. }, { // 8 join table without window
  2353. sql: `SELECT id1 FROM src1 INNER JOIN tableInPlanner on src1.id1 = tableInPlanner.id and src1.temp > 20 and hum < 60 WHERE src1.id1 > 111`,
  2354. p: ProjectPlan{
  2355. baseLogicalPlan: baseLogicalPlan{
  2356. children: []LogicalPlan{
  2357. JoinPlan{
  2358. baseLogicalPlan: baseLogicalPlan{
  2359. children: []LogicalPlan{
  2360. JoinAlignPlan{
  2361. baseLogicalPlan: baseLogicalPlan{
  2362. children: []LogicalPlan{
  2363. FilterPlan{
  2364. baseLogicalPlan: baseLogicalPlan{
  2365. children: []LogicalPlan{
  2366. DataSourcePlan{
  2367. name: "src1",
  2368. streamFields: map[string]*ast.JsonStreamField{
  2369. "hum": nil,
  2370. "id1": nil,
  2371. "temp": nil,
  2372. },
  2373. isSchemaless: true,
  2374. streamStmt: streams["src1"],
  2375. metaFields: []string{},
  2376. }.Init(),
  2377. },
  2378. },
  2379. condition: &ast.BinaryExpr{
  2380. RHS: &ast.BinaryExpr{
  2381. OP: ast.GT,
  2382. LHS: &ast.FieldRef{Name: "temp", StreamName: "src1"},
  2383. RHS: &ast.IntegerLiteral{Val: 20},
  2384. },
  2385. OP: ast.AND,
  2386. LHS: &ast.BinaryExpr{
  2387. OP: ast.GT,
  2388. LHS: &ast.FieldRef{Name: "id1", StreamName: "src1"},
  2389. RHS: &ast.IntegerLiteral{Val: 111},
  2390. },
  2391. },
  2392. }.Init(),
  2393. DataSourcePlan{
  2394. name: "tableInPlanner",
  2395. streamFields: map[string]*ast.JsonStreamField{
  2396. "hum": {
  2397. Type: "bigint",
  2398. },
  2399. "id": {
  2400. Type: "bigint",
  2401. },
  2402. },
  2403. streamStmt: streams["tableInPlanner"],
  2404. metaFields: []string{},
  2405. }.Init(),
  2406. },
  2407. },
  2408. Emitters: []string{"tableInPlanner"},
  2409. }.Init(),
  2410. },
  2411. },
  2412. from: &ast.Table{
  2413. Name: "src1",
  2414. },
  2415. joins: []ast.Join{
  2416. {
  2417. Name: "tableInPlanner",
  2418. Alias: "",
  2419. JoinType: ast.INNER_JOIN,
  2420. Expr: &ast.BinaryExpr{
  2421. OP: ast.AND,
  2422. LHS: &ast.BinaryExpr{
  2423. LHS: &ast.FieldRef{Name: "id1", StreamName: "src1"},
  2424. OP: ast.EQ,
  2425. RHS: &ast.FieldRef{Name: "id", StreamName: "tableInPlanner"},
  2426. },
  2427. RHS: &ast.BinaryExpr{
  2428. OP: ast.LT,
  2429. LHS: &ast.FieldRef{Name: "hum", StreamName: ast.DefaultStream},
  2430. RHS: &ast.IntegerLiteral{Val: 60},
  2431. },
  2432. },
  2433. },
  2434. },
  2435. }.Init(),
  2436. },
  2437. },
  2438. fields: []ast.Field{
  2439. {
  2440. Expr: &ast.FieldRef{Name: "id1", StreamName: ast.DefaultStream},
  2441. Name: "id1",
  2442. AName: "",
  2443. },
  2444. },
  2445. isAggregate: false,
  2446. sendMeta: false,
  2447. }.Init(),
  2448. }, { // 9 join table with window
  2449. sql: `SELECT id1 FROM src1 INNER JOIN tableInPlanner on src1.id1 = tableInPlanner.id and src1.temp > 20 and tableInPlanner.hum < 60 WHERE src1.id1 > 111 GROUP BY TUMBLINGWINDOW(ss, 10)`,
  2450. p: ProjectPlan{
  2451. baseLogicalPlan: baseLogicalPlan{
  2452. children: []LogicalPlan{
  2453. JoinPlan{
  2454. baseLogicalPlan: baseLogicalPlan{
  2455. children: []LogicalPlan{
  2456. JoinAlignPlan{
  2457. baseLogicalPlan: baseLogicalPlan{
  2458. children: []LogicalPlan{
  2459. WindowPlan{
  2460. baseLogicalPlan: baseLogicalPlan{
  2461. children: []LogicalPlan{
  2462. DataSourcePlan{
  2463. name: "src1",
  2464. streamFields: map[string]*ast.JsonStreamField{
  2465. "id1": nil,
  2466. "temp": nil,
  2467. },
  2468. isSchemaless: true,
  2469. streamStmt: streams["src1"],
  2470. metaFields: []string{},
  2471. }.Init(),
  2472. },
  2473. },
  2474. condition: nil,
  2475. wtype: ast.TUMBLING_WINDOW,
  2476. length: 10,
  2477. timeUnit: ast.SS,
  2478. interval: 0,
  2479. limit: 0,
  2480. }.Init(),
  2481. DataSourcePlan{
  2482. name: "tableInPlanner",
  2483. streamFields: map[string]*ast.JsonStreamField{
  2484. "hum": {
  2485. Type: "bigint",
  2486. },
  2487. "id": {
  2488. Type: "bigint",
  2489. },
  2490. },
  2491. streamStmt: streams["tableInPlanner"],
  2492. metaFields: []string{},
  2493. }.Init(),
  2494. },
  2495. },
  2496. Emitters: []string{"tableInPlanner"},
  2497. }.Init(),
  2498. },
  2499. },
  2500. from: &ast.Table{
  2501. Name: "src1",
  2502. },
  2503. joins: []ast.Join{
  2504. {
  2505. Name: "tableInPlanner",
  2506. Alias: "",
  2507. JoinType: ast.INNER_JOIN,
  2508. Expr: &ast.BinaryExpr{
  2509. OP: ast.AND,
  2510. LHS: &ast.BinaryExpr{
  2511. LHS: &ast.FieldRef{Name: "id1", StreamName: "src1"},
  2512. OP: ast.EQ,
  2513. RHS: &ast.FieldRef{Name: "id", StreamName: "tableInPlanner"},
  2514. },
  2515. RHS: &ast.BinaryExpr{
  2516. RHS: &ast.BinaryExpr{
  2517. OP: ast.AND,
  2518. LHS: &ast.BinaryExpr{
  2519. OP: ast.GT,
  2520. LHS: &ast.FieldRef{Name: "temp", StreamName: "src1"},
  2521. RHS: &ast.IntegerLiteral{Val: 20},
  2522. },
  2523. RHS: &ast.BinaryExpr{
  2524. OP: ast.LT,
  2525. LHS: &ast.FieldRef{Name: "hum", StreamName: "tableInPlanner"},
  2526. RHS: &ast.IntegerLiteral{Val: 60},
  2527. },
  2528. },
  2529. OP: ast.AND,
  2530. LHS: &ast.BinaryExpr{
  2531. OP: ast.GT,
  2532. LHS: &ast.FieldRef{Name: "id1", StreamName: "src1"},
  2533. RHS: &ast.IntegerLiteral{Val: 111},
  2534. },
  2535. },
  2536. },
  2537. },
  2538. },
  2539. }.Init(),
  2540. },
  2541. },
  2542. fields: []ast.Field{
  2543. {
  2544. Expr: &ast.FieldRef{Name: "id1", StreamName: ast.DefaultStream},
  2545. Name: "id1",
  2546. AName: "",
  2547. },
  2548. },
  2549. isAggregate: false,
  2550. sendMeta: false,
  2551. }.Init(),
  2552. }, { // 10 meta
  2553. sql: `SELECT temp, meta(id) AS eid,meta(Humidity->Device) AS hdevice FROM src1 WHERE meta(device)="demo2"`,
  2554. p: ProjectPlan{
  2555. baseLogicalPlan: baseLogicalPlan{
  2556. children: []LogicalPlan{
  2557. FilterPlan{
  2558. baseLogicalPlan: baseLogicalPlan{
  2559. children: []LogicalPlan{
  2560. DataSourcePlan{
  2561. name: "src1",
  2562. streamFields: map[string]*ast.JsonStreamField{
  2563. "temp": nil,
  2564. },
  2565. isSchemaless: true,
  2566. streamStmt: streams["src1"],
  2567. metaFields: []string{"Humidity", "device", "id"},
  2568. }.Init(),
  2569. },
  2570. },
  2571. condition: &ast.BinaryExpr{
  2572. LHS: &ast.Call{
  2573. Name: "meta",
  2574. FuncId: 2,
  2575. Args: []ast.Expr{&ast.MetaRef{
  2576. Name: "device",
  2577. StreamName: ast.DefaultStream,
  2578. }},
  2579. },
  2580. OP: ast.EQ,
  2581. RHS: &ast.StringLiteral{
  2582. Val: "demo2",
  2583. },
  2584. },
  2585. }.Init(),
  2586. },
  2587. },
  2588. fields: []ast.Field{
  2589. {
  2590. Expr: &ast.FieldRef{Name: "temp", StreamName: "src1"},
  2591. Name: "temp",
  2592. AName: "",
  2593. }, {
  2594. Expr: &ast.FieldRef{Name: "eid", StreamName: ast.AliasStream, AliasRef: ast.MockAliasRef(
  2595. &ast.Call{Name: "meta", FuncId: 0, Args: []ast.Expr{&ast.MetaRef{
  2596. Name: "id",
  2597. StreamName: ast.DefaultStream,
  2598. }}},
  2599. []ast.StreamName{},
  2600. nil,
  2601. )},
  2602. Name: "meta",
  2603. AName: "eid",
  2604. }, {
  2605. Expr: &ast.FieldRef{Name: "hdevice", StreamName: ast.AliasStream, AliasRef: ast.MockAliasRef(
  2606. &ast.Call{Name: "meta", FuncId: 1, Args: []ast.Expr{
  2607. &ast.BinaryExpr{
  2608. OP: ast.ARROW,
  2609. LHS: &ast.MetaRef{Name: "Humidity", StreamName: ast.DefaultStream},
  2610. RHS: &ast.JsonFieldRef{Name: "Device"},
  2611. },
  2612. }},
  2613. []ast.StreamName{},
  2614. nil,
  2615. )},
  2616. Name: "meta",
  2617. AName: "hdevice",
  2618. },
  2619. },
  2620. isAggregate: false,
  2621. sendMeta: false,
  2622. }.Init(),
  2623. }, { // 11 join with same name field and aliased
  2624. sql: `SELECT src2.hum AS hum1, tableInPlanner.hum AS hum2 FROM src2 INNER JOIN tableInPlanner on id2 = id WHERE hum1 > hum2`,
  2625. p: ProjectPlan{
  2626. baseLogicalPlan: baseLogicalPlan{
  2627. children: []LogicalPlan{
  2628. JoinPlan{
  2629. baseLogicalPlan: baseLogicalPlan{
  2630. children: []LogicalPlan{
  2631. JoinAlignPlan{
  2632. baseLogicalPlan: baseLogicalPlan{
  2633. children: []LogicalPlan{
  2634. DataSourcePlan{
  2635. name: "src2",
  2636. streamFields: map[string]*ast.JsonStreamField{
  2637. "hum": nil,
  2638. "id": nil,
  2639. "id2": nil,
  2640. },
  2641. isSchemaless: true,
  2642. streamStmt: streams["src2"],
  2643. metaFields: []string{},
  2644. }.Init(),
  2645. DataSourcePlan{
  2646. name: "tableInPlanner",
  2647. streamFields: map[string]*ast.JsonStreamField{
  2648. "hum": {
  2649. Type: "bigint",
  2650. },
  2651. "id": {
  2652. Type: "bigint",
  2653. },
  2654. },
  2655. streamStmt: streams["tableInPlanner"],
  2656. metaFields: []string{},
  2657. }.Init(),
  2658. },
  2659. },
  2660. Emitters: []string{"tableInPlanner"},
  2661. }.Init(),
  2662. },
  2663. },
  2664. from: &ast.Table{
  2665. Name: "src2",
  2666. },
  2667. joins: []ast.Join{
  2668. {
  2669. Name: "tableInPlanner",
  2670. Alias: "",
  2671. JoinType: ast.INNER_JOIN,
  2672. Expr: &ast.BinaryExpr{
  2673. RHS: &ast.BinaryExpr{
  2674. OP: ast.EQ,
  2675. LHS: &ast.FieldRef{Name: "id2", StreamName: ast.DefaultStream},
  2676. RHS: &ast.FieldRef{Name: "id", StreamName: ast.DefaultStream},
  2677. },
  2678. OP: ast.AND,
  2679. LHS: &ast.BinaryExpr{
  2680. OP: ast.GT,
  2681. LHS: &ast.FieldRef{Name: "hum1", StreamName: ast.AliasStream, AliasRef: ast.MockAliasRef(
  2682. &ast.FieldRef{
  2683. Name: "hum",
  2684. StreamName: "src2",
  2685. },
  2686. []ast.StreamName{"src2"},
  2687. &boolFalse,
  2688. )},
  2689. RHS: &ast.FieldRef{Name: "hum2", StreamName: ast.AliasStream, AliasRef: ast.MockAliasRef(
  2690. &ast.FieldRef{
  2691. Name: "hum",
  2692. StreamName: "tableInPlanner",
  2693. },
  2694. []ast.StreamName{"tableInPlanner"},
  2695. &boolFalse,
  2696. )},
  2697. },
  2698. },
  2699. },
  2700. },
  2701. }.Init(),
  2702. },
  2703. },
  2704. fields: []ast.Field{
  2705. {
  2706. Expr: &ast.FieldRef{Name: "hum1", StreamName: ast.AliasStream, AliasRef: ast.MockAliasRef(
  2707. &ast.FieldRef{
  2708. Name: "hum",
  2709. StreamName: "src2",
  2710. },
  2711. []ast.StreamName{"src2"},
  2712. &boolFalse,
  2713. )},
  2714. Name: "hum",
  2715. AName: "hum1",
  2716. }, {
  2717. Expr: &ast.FieldRef{Name: "hum2", StreamName: ast.AliasStream, AliasRef: ast.MockAliasRef(
  2718. &ast.FieldRef{
  2719. Name: "hum",
  2720. StreamName: "tableInPlanner",
  2721. },
  2722. []ast.StreamName{"tableInPlanner"},
  2723. &boolFalse,
  2724. )},
  2725. Name: "hum",
  2726. AName: "hum2",
  2727. },
  2728. },
  2729. isAggregate: false,
  2730. sendMeta: false,
  2731. }.Init(),
  2732. }, { // 12
  2733. sql: `SELECT name->first, name->last FROM src1`,
  2734. p: ProjectPlan{
  2735. baseLogicalPlan: baseLogicalPlan{
  2736. children: []LogicalPlan{
  2737. DataSourcePlan{
  2738. baseLogicalPlan: baseLogicalPlan{},
  2739. name: "src1",
  2740. streamFields: map[string]*ast.JsonStreamField{
  2741. "name": nil,
  2742. },
  2743. isSchemaless: true,
  2744. streamStmt: streams["src1"],
  2745. metaFields: []string{},
  2746. }.Init(),
  2747. },
  2748. },
  2749. fields: []ast.Field{
  2750. {
  2751. Expr: &ast.BinaryExpr{
  2752. OP: ast.ARROW,
  2753. LHS: &ast.FieldRef{StreamName: "src1", Name: "name"},
  2754. RHS: &ast.JsonFieldRef{Name: "first"},
  2755. },
  2756. Name: "kuiper_field_0",
  2757. AName: "",
  2758. }, {
  2759. Expr: &ast.BinaryExpr{
  2760. OP: ast.ARROW,
  2761. LHS: &ast.FieldRef{StreamName: "src1", Name: "name"},
  2762. RHS: &ast.JsonFieldRef{Name: "last"},
  2763. },
  2764. Name: "kuiper_field_1",
  2765. AName: "",
  2766. },
  2767. },
  2768. isAggregate: false,
  2769. sendMeta: false,
  2770. }.Init(),
  2771. },
  2772. }
  2773. fmt.Printf("The test bucket size is %d.\n\n", len(tests))
  2774. for i, tt := range tests {
  2775. stmt, err := xsql.NewParser(strings.NewReader(tt.sql)).Parse()
  2776. if err != nil {
  2777. t.Errorf("%d. %q: error compile sql: %s\n", i, tt.sql, err)
  2778. continue
  2779. }
  2780. p, err := createLogicalPlan(stmt, &api.RuleOption{
  2781. IsEventTime: false,
  2782. LateTol: 0,
  2783. Concurrency: 0,
  2784. BufferLength: 0,
  2785. SendMetaToSink: false,
  2786. Qos: 0,
  2787. CheckpointInterval: 0,
  2788. SendError: true,
  2789. }, kv)
  2790. if !reflect.DeepEqual(tt.err, testx.Errstring(err)) {
  2791. t.Errorf("%d. %q: error mismatch:\n exp=%s\n got=%s\n\n", i, tt.sql, tt.err, err)
  2792. } else if !reflect.DeepEqual(tt.p, p) {
  2793. t.Errorf("%d. %q\n\nstmt mismatch:\n\nexp=%#v\n\ngot=%#v\n\n", i, tt.sql, render.AsCode(tt.p), render.AsCode(p))
  2794. }
  2795. }
  2796. }
  2797. func Test_createLogicalPlan4Lookup(t *testing.T) {
  2798. kv, err := store.GetKV("stream")
  2799. if err != nil {
  2800. t.Error(err)
  2801. return
  2802. }
  2803. streamSqls := map[string]string{
  2804. "src1": `CREATE STREAM src1 () WITH (DATASOURCE="src1", FORMAT="json", KEY="ts");`,
  2805. "table1": `CREATE TABLE table1 () WITH (DATASOURCE="table1",TYPE="sql", KIND="lookup");`,
  2806. "table2": `CREATE TABLE table2 () WITH (DATASOURCE="table2",TYPE="sql", KIND="lookup");`,
  2807. }
  2808. types := map[string]ast.StreamType{
  2809. "src1": ast.TypeStream,
  2810. "table1": ast.TypeTable,
  2811. "table2": ast.TypeTable,
  2812. }
  2813. for name, sql := range streamSqls {
  2814. s, err := json.Marshal(&xsql.StreamInfo{
  2815. StreamType: types[name],
  2816. Statement: sql,
  2817. })
  2818. if err != nil {
  2819. t.Error(err)
  2820. t.Fail()
  2821. }
  2822. err = kv.Set(name, string(s))
  2823. if err != nil {
  2824. t.Error(err)
  2825. t.Fail()
  2826. }
  2827. }
  2828. streams := make(map[string]*ast.StreamStmt)
  2829. for n := range streamSqls {
  2830. streamStmt, err := xsql.GetDataSource(kv, n)
  2831. if err != nil {
  2832. t.Errorf("fail to get stream %s, please check if stream is created", n)
  2833. return
  2834. }
  2835. streams[n] = streamStmt
  2836. }
  2837. tests := []struct {
  2838. sql string
  2839. p LogicalPlan
  2840. err string
  2841. }{
  2842. { // 0
  2843. sql: `SELECT src1.a, table1.b FROM src1 INNER JOIN table1 ON src1.id = table1.id`,
  2844. p: ProjectPlan{
  2845. baseLogicalPlan: baseLogicalPlan{
  2846. children: []LogicalPlan{
  2847. LookupPlan{
  2848. baseLogicalPlan: baseLogicalPlan{
  2849. children: []LogicalPlan{
  2850. DataSourcePlan{
  2851. baseLogicalPlan: baseLogicalPlan{},
  2852. name: "src1",
  2853. streamFields: map[string]*ast.JsonStreamField{
  2854. "a": nil,
  2855. },
  2856. isSchemaless: true,
  2857. streamStmt: streams["src1"],
  2858. metaFields: []string{},
  2859. }.Init(),
  2860. },
  2861. },
  2862. joinExpr: ast.Join{
  2863. Name: "table1",
  2864. Alias: "",
  2865. JoinType: ast.INNER_JOIN,
  2866. Expr: &ast.BinaryExpr{
  2867. OP: ast.EQ,
  2868. LHS: &ast.FieldRef{
  2869. StreamName: "src1",
  2870. Name: "id",
  2871. },
  2872. RHS: &ast.FieldRef{
  2873. StreamName: "table1",
  2874. Name: "id",
  2875. },
  2876. },
  2877. },
  2878. keys: []string{"id"},
  2879. fields: []string{"b"},
  2880. valvars: []ast.Expr{
  2881. &ast.FieldRef{
  2882. StreamName: "src1",
  2883. Name: "id",
  2884. },
  2885. },
  2886. options: &ast.Options{
  2887. DATASOURCE: "table1",
  2888. TYPE: "sql",
  2889. KIND: "lookup",
  2890. },
  2891. conditions: nil,
  2892. }.Init(),
  2893. },
  2894. },
  2895. fields: []ast.Field{
  2896. {
  2897. Expr: &ast.FieldRef{
  2898. StreamName: "src1",
  2899. Name: "a",
  2900. },
  2901. Name: "a",
  2902. AName: "",
  2903. },
  2904. {
  2905. Expr: &ast.FieldRef{
  2906. StreamName: "table1",
  2907. Name: "b",
  2908. },
  2909. Name: "b",
  2910. AName: "",
  2911. },
  2912. },
  2913. isAggregate: false,
  2914. sendMeta: false,
  2915. }.Init(),
  2916. },
  2917. { // 1
  2918. sql: `SELECT src1.a, table1.* FROM src1 INNER JOIN table1 ON table1.b > 20 AND src1.c < 40 AND src1.id = table1.id`,
  2919. p: ProjectPlan{
  2920. baseLogicalPlan: baseLogicalPlan{
  2921. children: []LogicalPlan{
  2922. FilterPlan{
  2923. baseLogicalPlan: baseLogicalPlan{
  2924. children: []LogicalPlan{
  2925. LookupPlan{
  2926. baseLogicalPlan: baseLogicalPlan{
  2927. children: []LogicalPlan{
  2928. FilterPlan{
  2929. baseLogicalPlan: baseLogicalPlan{
  2930. children: []LogicalPlan{
  2931. DataSourcePlan{
  2932. baseLogicalPlan: baseLogicalPlan{},
  2933. name: "src1",
  2934. streamFields: map[string]*ast.JsonStreamField{
  2935. "a": nil,
  2936. },
  2937. isSchemaless: true,
  2938. streamStmt: streams["src1"],
  2939. metaFields: []string{},
  2940. }.Init(),
  2941. },
  2942. },
  2943. condition: &ast.BinaryExpr{
  2944. OP: ast.LT,
  2945. LHS: &ast.FieldRef{
  2946. StreamName: "src1",
  2947. Name: "c",
  2948. },
  2949. RHS: &ast.IntegerLiteral{Val: 40},
  2950. },
  2951. }.Init(),
  2952. },
  2953. },
  2954. joinExpr: ast.Join{
  2955. Name: "table1",
  2956. Alias: "",
  2957. JoinType: ast.INNER_JOIN,
  2958. Expr: &ast.BinaryExpr{
  2959. OP: ast.AND,
  2960. RHS: &ast.BinaryExpr{
  2961. OP: ast.EQ,
  2962. LHS: &ast.FieldRef{
  2963. StreamName: "src1",
  2964. Name: "id",
  2965. },
  2966. RHS: &ast.FieldRef{
  2967. StreamName: "table1",
  2968. Name: "id",
  2969. },
  2970. },
  2971. LHS: &ast.BinaryExpr{
  2972. OP: ast.AND,
  2973. LHS: &ast.BinaryExpr{
  2974. OP: ast.GT,
  2975. LHS: &ast.FieldRef{
  2976. StreamName: "table1",
  2977. Name: "b",
  2978. },
  2979. RHS: &ast.IntegerLiteral{Val: 20},
  2980. },
  2981. RHS: &ast.BinaryExpr{
  2982. OP: ast.LT,
  2983. LHS: &ast.FieldRef{
  2984. StreamName: "src1",
  2985. Name: "c",
  2986. },
  2987. RHS: &ast.IntegerLiteral{Val: 40},
  2988. },
  2989. },
  2990. },
  2991. },
  2992. keys: []string{"id"},
  2993. valvars: []ast.Expr{
  2994. &ast.FieldRef{
  2995. StreamName: "src1",
  2996. Name: "id",
  2997. },
  2998. },
  2999. options: &ast.Options{
  3000. DATASOURCE: "table1",
  3001. TYPE: "sql",
  3002. KIND: "lookup",
  3003. },
  3004. conditions: &ast.BinaryExpr{
  3005. OP: ast.AND,
  3006. LHS: &ast.BinaryExpr{
  3007. OP: ast.GT,
  3008. LHS: &ast.FieldRef{
  3009. StreamName: "table1",
  3010. Name: "b",
  3011. },
  3012. RHS: &ast.IntegerLiteral{Val: 20},
  3013. },
  3014. RHS: &ast.BinaryExpr{
  3015. OP: ast.LT,
  3016. LHS: &ast.FieldRef{
  3017. StreamName: "src1",
  3018. Name: "c",
  3019. },
  3020. RHS: &ast.IntegerLiteral{Val: 40},
  3021. },
  3022. },
  3023. }.Init(),
  3024. },
  3025. },
  3026. condition: &ast.BinaryExpr{
  3027. OP: ast.GT,
  3028. LHS: &ast.FieldRef{
  3029. StreamName: "table1",
  3030. Name: "b",
  3031. },
  3032. RHS: &ast.IntegerLiteral{Val: 20},
  3033. },
  3034. }.Init(),
  3035. },
  3036. },
  3037. fields: []ast.Field{
  3038. {
  3039. Expr: &ast.FieldRef{
  3040. StreamName: "src1",
  3041. Name: "a",
  3042. },
  3043. Name: "a",
  3044. AName: "",
  3045. },
  3046. {
  3047. Expr: &ast.FieldRef{
  3048. StreamName: "table1",
  3049. Name: "*",
  3050. },
  3051. Name: "*",
  3052. AName: "",
  3053. },
  3054. },
  3055. isAggregate: false,
  3056. sendMeta: false,
  3057. }.Init(),
  3058. },
  3059. { // 2
  3060. sql: `SELECT src1.a, table1.b, table2.c FROM src1 INNER JOIN table1 ON src1.id = table1.id INNER JOIN table2 on table1.id = table2.id`,
  3061. p: ProjectPlan{
  3062. baseLogicalPlan: baseLogicalPlan{
  3063. children: []LogicalPlan{
  3064. LookupPlan{
  3065. baseLogicalPlan: baseLogicalPlan{
  3066. children: []LogicalPlan{
  3067. LookupPlan{
  3068. baseLogicalPlan: baseLogicalPlan{
  3069. children: []LogicalPlan{
  3070. DataSourcePlan{
  3071. baseLogicalPlan: baseLogicalPlan{},
  3072. name: "src1",
  3073. streamFields: map[string]*ast.JsonStreamField{
  3074. "a": nil,
  3075. },
  3076. isSchemaless: true,
  3077. streamStmt: streams["src1"],
  3078. metaFields: []string{},
  3079. }.Init(),
  3080. },
  3081. },
  3082. joinExpr: ast.Join{
  3083. Name: "table1",
  3084. Alias: "",
  3085. JoinType: ast.INNER_JOIN,
  3086. Expr: &ast.BinaryExpr{
  3087. OP: ast.EQ,
  3088. LHS: &ast.FieldRef{
  3089. StreamName: "src1",
  3090. Name: "id",
  3091. },
  3092. RHS: &ast.FieldRef{
  3093. StreamName: "table1",
  3094. Name: "id",
  3095. },
  3096. },
  3097. },
  3098. keys: []string{"id"},
  3099. fields: []string{"b"},
  3100. valvars: []ast.Expr{
  3101. &ast.FieldRef{
  3102. StreamName: "src1",
  3103. Name: "id",
  3104. },
  3105. },
  3106. options: &ast.Options{
  3107. DATASOURCE: "table1",
  3108. TYPE: "sql",
  3109. KIND: "lookup",
  3110. },
  3111. conditions: nil,
  3112. }.Init(),
  3113. },
  3114. },
  3115. joinExpr: ast.Join{
  3116. Name: "table2",
  3117. Alias: "",
  3118. JoinType: ast.INNER_JOIN,
  3119. Expr: &ast.BinaryExpr{
  3120. OP: ast.EQ,
  3121. LHS: &ast.FieldRef{
  3122. StreamName: "table1",
  3123. Name: "id",
  3124. },
  3125. RHS: &ast.FieldRef{
  3126. StreamName: "table2",
  3127. Name: "id",
  3128. },
  3129. },
  3130. },
  3131. keys: []string{"id"},
  3132. fields: []string{"c"},
  3133. valvars: []ast.Expr{
  3134. &ast.FieldRef{
  3135. StreamName: "table1",
  3136. Name: "id",
  3137. },
  3138. },
  3139. options: &ast.Options{
  3140. DATASOURCE: "table2",
  3141. TYPE: "sql",
  3142. KIND: "lookup",
  3143. },
  3144. }.Init(),
  3145. },
  3146. },
  3147. fields: []ast.Field{
  3148. {
  3149. Expr: &ast.FieldRef{
  3150. StreamName: "src1",
  3151. Name: "a",
  3152. },
  3153. Name: "a",
  3154. AName: "",
  3155. },
  3156. {
  3157. Expr: &ast.FieldRef{
  3158. StreamName: "table1",
  3159. Name: "b",
  3160. },
  3161. Name: "b",
  3162. AName: "",
  3163. },
  3164. {
  3165. Expr: &ast.FieldRef{
  3166. StreamName: "table2",
  3167. Name: "c",
  3168. },
  3169. Name: "c",
  3170. AName: "",
  3171. },
  3172. },
  3173. isAggregate: false,
  3174. sendMeta: false,
  3175. }.Init(),
  3176. },
  3177. { // 3
  3178. sql: `SELECT * FROM src1 INNER JOIN table1 ON src1.id = table1.id GROUP BY TUMBLINGWINDOW(ss, 10)`,
  3179. p: ProjectPlan{
  3180. baseLogicalPlan: baseLogicalPlan{
  3181. children: []LogicalPlan{
  3182. LookupPlan{
  3183. baseLogicalPlan: baseLogicalPlan{
  3184. children: []LogicalPlan{
  3185. WindowPlan{
  3186. baseLogicalPlan: baseLogicalPlan{
  3187. children: []LogicalPlan{
  3188. DataSourcePlan{
  3189. baseLogicalPlan: baseLogicalPlan{},
  3190. name: "src1",
  3191. streamStmt: streams["src1"],
  3192. streamFields: map[string]*ast.JsonStreamField{},
  3193. metaFields: []string{},
  3194. isWildCard: true,
  3195. isSchemaless: true,
  3196. }.Init(),
  3197. },
  3198. },
  3199. condition: nil,
  3200. wtype: ast.TUMBLING_WINDOW,
  3201. length: 10,
  3202. timeUnit: ast.SS,
  3203. interval: 0,
  3204. limit: 0,
  3205. }.Init(),
  3206. },
  3207. },
  3208. joinExpr: ast.Join{
  3209. Name: "table1",
  3210. Alias: "",
  3211. JoinType: ast.INNER_JOIN,
  3212. Expr: &ast.BinaryExpr{
  3213. OP: ast.EQ,
  3214. LHS: &ast.FieldRef{
  3215. StreamName: "src1",
  3216. Name: "id",
  3217. },
  3218. RHS: &ast.FieldRef{
  3219. StreamName: "table1",
  3220. Name: "id",
  3221. },
  3222. },
  3223. },
  3224. keys: []string{"id"},
  3225. valvars: []ast.Expr{
  3226. &ast.FieldRef{
  3227. StreamName: "src1",
  3228. Name: "id",
  3229. },
  3230. },
  3231. options: &ast.Options{
  3232. DATASOURCE: "table1",
  3233. TYPE: "sql",
  3234. KIND: "lookup",
  3235. },
  3236. conditions: nil,
  3237. }.Init(),
  3238. },
  3239. },
  3240. fields: []ast.Field{
  3241. {
  3242. Expr: &ast.Wildcard{Token: ast.ASTERISK},
  3243. Name: "*",
  3244. AName: "",
  3245. },
  3246. },
  3247. isAggregate: false,
  3248. sendMeta: false,
  3249. }.Init(),
  3250. },
  3251. }
  3252. for i, tt := range tests {
  3253. stmt, err := xsql.NewParser(strings.NewReader(tt.sql)).Parse()
  3254. if err != nil {
  3255. t.Errorf("%d. %q: error compile sql: %s\n", i, tt.sql, err)
  3256. continue
  3257. }
  3258. p, err := createLogicalPlan(stmt, &api.RuleOption{
  3259. IsEventTime: false,
  3260. LateTol: 0,
  3261. Concurrency: 0,
  3262. BufferLength: 0,
  3263. SendMetaToSink: false,
  3264. Qos: 0,
  3265. CheckpointInterval: 0,
  3266. SendError: true,
  3267. }, kv)
  3268. if !reflect.DeepEqual(tt.err, testx.Errstring(err)) {
  3269. t.Errorf("%d. %q: error mismatch:\n exp=%s\n got=%s\n\n", i, tt.sql, tt.err, err)
  3270. } else if !reflect.DeepEqual(tt.p, p) {
  3271. t.Errorf("%d. %q\n\nstmt mismatch:\n\nexp=%#v\n\ngot=%#v\n\n", i, tt.sql, render.AsCode(tt.p), render.AsCode(p))
  3272. }
  3273. }
  3274. }
  3275. func TestTransformSourceNode(t *testing.T) {
  3276. schema := map[string]*ast.JsonStreamField{
  3277. "a": {
  3278. Type: "bigint",
  3279. },
  3280. }
  3281. testCases := []struct {
  3282. name string
  3283. plan *DataSourcePlan
  3284. node *node.SourceNode
  3285. }{
  3286. {
  3287. name: "normal source node",
  3288. plan: &DataSourcePlan{
  3289. name: "test",
  3290. streamStmt: &ast.StreamStmt{
  3291. StreamType: ast.TypeStream,
  3292. Options: &ast.Options{
  3293. TYPE: "file",
  3294. },
  3295. },
  3296. streamFields: nil,
  3297. allMeta: false,
  3298. metaFields: []string{},
  3299. iet: false,
  3300. isBinary: false,
  3301. },
  3302. node: node.NewSourceNode("test", ast.TypeStream, nil, &ast.Options{
  3303. TYPE: "file",
  3304. }, false, nil),
  3305. },
  3306. {
  3307. name: "schema source node",
  3308. plan: &DataSourcePlan{
  3309. name: "test",
  3310. streamStmt: &ast.StreamStmt{
  3311. StreamType: ast.TypeStream,
  3312. Options: &ast.Options{
  3313. TYPE: "file",
  3314. },
  3315. },
  3316. streamFields: schema,
  3317. allMeta: false,
  3318. metaFields: []string{},
  3319. iet: false,
  3320. isBinary: false,
  3321. },
  3322. node: node.NewSourceNode("test", ast.TypeStream, nil, &ast.Options{
  3323. TYPE: "file",
  3324. }, false, schema),
  3325. },
  3326. }
  3327. for _, tc := range testCases {
  3328. t.Run(tc.name, func(t *testing.T) {
  3329. sourceNode, err := transformSourceNode(tc.plan, nil, &api.RuleOption{})
  3330. if err != nil {
  3331. t.Errorf("unexpected error: %v", err)
  3332. return
  3333. }
  3334. if !reflect.DeepEqual(sourceNode, tc.node) {
  3335. t.Errorf("unexpected result: got %v, want %v", sourceNode, tc.node)
  3336. }
  3337. })
  3338. }
  3339. }