metadata.go 21 KB

123456789101112131415161718192021222324252627282930313233343536373839404142434445464748495051525354555657585960616263646566676869707172737475767778798081828384858687888990919293949596979899100101102103104105106107108109110111112113114115116117118119120121122123124125126127128129130131132133134135136137138139140141142143144145146147148149150151152153154155156157158159160161162163164165166167168169170171172173174175176177178179180181182183184185186187188189190191192193194195196197198199200201202203204205206207208209210211212213214215216217218219220221222223224225226227228229230231232233234235236237238239240241242243244245246247248249250251252253254255256257258259260261262263264265266267268269270271272273274275276277278279280281282283284285286287288289290291292293294295296297298299300301302303304305306307308309310311312313314315316317318319320321322323324325326327328329330331332333334335336337338339340341342343344345346347348349350351352353354355356357358359360361362363364365366367368369370371372373374375376377378379380381382383384385386387388389390391392393394395396397398399400401402403404405406407408409410411412413414415416417418419420421422423424425426427428429430431432433434435436437438439440441442443444445446447448449450451452453454455456457458459460461462463464465466467468469470471472473474475476477478479480481482483484485486487488489490491492493494495496497498499500501502503504505506507508509510511512513514515516517518519520521522523524525526527528529530531532533534535536537538539540541542543544545546547548549550551552553554555556557558559560561562563564565566567568569570571572573574575576577578579580581582583584585586587588589590591592593594595596597598599600601602603604605606607608609610611612613614615616617618619620621622623624625626627628629630631632633634635636637638639640641642643644645646647648649650651652653654655656657658659660661662663664665666667668669670671672673674675676677678679680681682683684685686687688689690691692693694695696697698699700701702703704705706707708709710711712713714715716717718719720721722723724725726727728729730731732733734735736737738739740741742743744745746747748749750751752753754755756757758759760761762763764765766767768769770771772773774775776777778779780781782783784785786787788789790791792793794795796797798799800801802803804805806807808809810811812813814815816817818819820821822823824825826827828829830831832833834835836837838839840841842843844845846847848849850851852853854855856857858859860861862863864865866867868869870871872873874875
  1. // Copyright (c) 2015 The gocql Authors. All rights reserved.
  2. // Use of this source code is governed by a BSD-style
  3. // license that can be found in the LICENSE file.
  4. package gocql
  5. import (
  6. "encoding/hex"
  7. "encoding/json"
  8. "fmt"
  9. "log"
  10. "strconv"
  11. "strings"
  12. "sync"
  13. )
  14. // schema metadata for a keyspace
  15. type KeyspaceMetadata struct {
  16. Name string
  17. DurableWrites bool
  18. StrategyClass string
  19. StrategyOptions map[string]interface{}
  20. Tables map[string]*TableMetadata
  21. }
  22. // schema metadata for a table (a.k.a. column family)
  23. type TableMetadata struct {
  24. Keyspace string
  25. Name string
  26. KeyValidator string
  27. Comparator string
  28. DefaultValidator string
  29. KeyAliases []string
  30. ColumnAliases []string
  31. ValueAlias string
  32. PartitionKey []*ColumnMetadata
  33. ClusteringColumns []*ColumnMetadata
  34. Columns map[string]*ColumnMetadata
  35. OrderedColumns []string
  36. }
  37. // schema metadata for a column
  38. type ColumnMetadata struct {
  39. Keyspace string
  40. Table string
  41. Name string
  42. ComponentIndex int
  43. Kind string
  44. Validator string
  45. Type TypeInfo
  46. Order ColumnOrder
  47. Index ColumnIndexMetadata
  48. }
  49. // the ordering of the column with regard to its comparator
  50. type ColumnOrder bool
  51. const (
  52. ASC ColumnOrder = false
  53. DESC = true
  54. )
  55. type ColumnIndexMetadata struct {
  56. Name string
  57. Type string
  58. Options map[string]interface{}
  59. }
  60. // Column kind values
  61. const (
  62. PARTITION_KEY = "partition_key"
  63. CLUSTERING_KEY = "clustering_key"
  64. REGULAR = "regular"
  65. COMPACT_VALUE = "compact_value"
  66. )
  67. // default alias values
  68. const (
  69. DEFAULT_KEY_ALIAS = "key"
  70. DEFAULT_COLUMN_ALIAS = "column"
  71. DEFAULT_VALUE_ALIAS = "value"
  72. )
  73. // queries the cluster for schema information for a specific keyspace
  74. type schemaDescriber struct {
  75. session *Session
  76. mu sync.Mutex
  77. cache map[string]*KeyspaceMetadata
  78. }
  79. // creates a session bound schema describer which will query and cache
  80. // keyspace metadata
  81. func newSchemaDescriber(session *Session) *schemaDescriber {
  82. return &schemaDescriber{
  83. session: session,
  84. cache: map[string]*KeyspaceMetadata{},
  85. }
  86. }
  87. // returns the cached KeyspaceMetadata held by the describer for the named
  88. // keyspace.
  89. func (s *schemaDescriber) getSchema(keyspaceName string) (*KeyspaceMetadata, error) {
  90. s.mu.Lock()
  91. defer s.mu.Unlock()
  92. // TODO handle schema change events
  93. metadata, found := s.cache[keyspaceName]
  94. if !found {
  95. // refresh the cache for this keyspace
  96. err := s.refreshSchema(keyspaceName)
  97. if err != nil {
  98. return nil, err
  99. }
  100. metadata = s.cache[keyspaceName]
  101. }
  102. return metadata, nil
  103. }
  104. // forcibly updates the current KeyspaceMetadata held by the schema describer
  105. // for a given named keyspace.
  106. func (s *schemaDescriber) refreshSchema(keyspaceName string) error {
  107. var err error
  108. // query the system keyspace for schema data
  109. // TODO retrieve concurrently
  110. keyspace, err := getKeyspaceMetadata(s.session, keyspaceName)
  111. if err != nil {
  112. return err
  113. }
  114. tables, err := getTableMetadata(s.session, keyspaceName)
  115. if err != nil {
  116. return err
  117. }
  118. columns, err := getColumnMetadata(s.session, keyspaceName)
  119. if err != nil {
  120. return err
  121. }
  122. // organize the schema data
  123. compileMetadata(s.session.cfg.ProtoVersion, keyspace, tables, columns)
  124. // update the cache
  125. s.cache[keyspaceName] = keyspace
  126. return nil
  127. }
  128. // "compiles" derived information about keyspace, table, and column metadata
  129. // for a keyspace from the basic queried metadata objects returned by
  130. // getKeyspaceMetadata, getTableMetadata, and getColumnMetadata respectively;
  131. // Links the metadata objects together and derives the column composition of
  132. // the partition key and clustering key for a table.
  133. func compileMetadata(
  134. protoVersion int,
  135. keyspace *KeyspaceMetadata,
  136. tables []TableMetadata,
  137. columns []ColumnMetadata,
  138. ) {
  139. keyspace.Tables = make(map[string]*TableMetadata)
  140. for i := range tables {
  141. tables[i].Columns = make(map[string]*ColumnMetadata)
  142. keyspace.Tables[tables[i].Name] = &tables[i]
  143. }
  144. // add columns from the schema data
  145. for i := range columns {
  146. // decode the validator for TypeInfo and order
  147. validatorParsed := parseType(columns[i].Validator)
  148. columns[i].Type = validatorParsed.types[0]
  149. columns[i].Order = ASC
  150. if validatorParsed.reversed[0] {
  151. columns[i].Order = DESC
  152. }
  153. table := keyspace.Tables[columns[i].Table]
  154. table.Columns[columns[i].Name] = &columns[i]
  155. table.OrderedColumns = append(table.OrderedColumns, columns[i].Name)
  156. }
  157. if protoVersion == 1 {
  158. compileV1Metadata(tables)
  159. } else {
  160. compileV2Metadata(tables)
  161. }
  162. }
  163. // Compiles derived information from TableMetadata which have had
  164. // ColumnMetadata added already. V1 protocol does not return as much
  165. // column metadata as V2+ (because V1 doesn't support the "type" column in the
  166. // system.schema_columns table) so determining PartitionKey and ClusterColumns
  167. // is more complex.
  168. func compileV1Metadata(tables []TableMetadata) {
  169. for i := range tables {
  170. table := &tables[i]
  171. // decode the key validator
  172. keyValidatorParsed := parseType(table.KeyValidator)
  173. // decode the comparator
  174. comparatorParsed := parseType(table.Comparator)
  175. // the partition key length is the same as the number of types in the
  176. // key validator
  177. table.PartitionKey = make([]*ColumnMetadata, len(keyValidatorParsed.types))
  178. // V1 protocol only returns "regular" columns from
  179. // system.schema_columns (there is no type field for columns)
  180. // so the alias information is used to
  181. // create the partition key and clustering columns
  182. // construct the partition key from the alias
  183. for i := range table.PartitionKey {
  184. var alias string
  185. if len(table.KeyAliases) > i {
  186. alias = table.KeyAliases[i]
  187. } else if i == 0 {
  188. alias = DEFAULT_KEY_ALIAS
  189. } else {
  190. alias = DEFAULT_KEY_ALIAS + strconv.Itoa(i+1)
  191. }
  192. column := &ColumnMetadata{
  193. Keyspace: table.Keyspace,
  194. Table: table.Name,
  195. Name: alias,
  196. Type: keyValidatorParsed.types[i],
  197. Kind: PARTITION_KEY,
  198. ComponentIndex: i,
  199. }
  200. table.PartitionKey[i] = column
  201. table.Columns[alias] = column
  202. }
  203. // determine the number of clustering columns
  204. size := len(comparatorParsed.types)
  205. if comparatorParsed.isComposite {
  206. if len(comparatorParsed.collections) != 0 ||
  207. (len(table.ColumnAliases) == size-1 &&
  208. comparatorParsed.types[size-1].Type() == TypeVarchar) {
  209. size = size - 1
  210. }
  211. } else {
  212. if !(len(table.ColumnAliases) != 0 || len(table.Columns) == 0) {
  213. size = 0
  214. }
  215. }
  216. table.ClusteringColumns = make([]*ColumnMetadata, size)
  217. for i := range table.ClusteringColumns {
  218. var alias string
  219. if len(table.ColumnAliases) > i {
  220. alias = table.ColumnAliases[i]
  221. } else if i == 0 {
  222. alias = DEFAULT_COLUMN_ALIAS
  223. } else {
  224. alias = DEFAULT_COLUMN_ALIAS + strconv.Itoa(i+1)
  225. }
  226. order := ASC
  227. if comparatorParsed.reversed[i] {
  228. order = DESC
  229. }
  230. column := &ColumnMetadata{
  231. Keyspace: table.Keyspace,
  232. Table: table.Name,
  233. Name: alias,
  234. Type: comparatorParsed.types[i],
  235. Order: order,
  236. Kind: CLUSTERING_KEY,
  237. ComponentIndex: i,
  238. }
  239. table.ClusteringColumns[i] = column
  240. table.Columns[alias] = column
  241. }
  242. if size != len(comparatorParsed.types)-1 {
  243. alias := DEFAULT_VALUE_ALIAS
  244. if len(table.ValueAlias) > 0 {
  245. alias = table.ValueAlias
  246. }
  247. // decode the default validator
  248. defaultValidatorParsed := parseType(table.DefaultValidator)
  249. column := &ColumnMetadata{
  250. Keyspace: table.Keyspace,
  251. Table: table.Name,
  252. Name: alias,
  253. Type: defaultValidatorParsed.types[0],
  254. Kind: REGULAR,
  255. }
  256. table.Columns[alias] = column
  257. }
  258. }
  259. }
  260. // The simpler compile case for V2+ protocol
  261. func compileV2Metadata(tables []TableMetadata) {
  262. for i := range tables {
  263. table := &tables[i]
  264. keyValidatorParsed := parseType(table.KeyValidator)
  265. table.PartitionKey = make([]*ColumnMetadata, len(keyValidatorParsed.types))
  266. clusteringColumnCount := componentColumnCountOfType(table.Columns, CLUSTERING_KEY)
  267. table.ClusteringColumns = make([]*ColumnMetadata, clusteringColumnCount)
  268. for _, columnName := range table.OrderedColumns {
  269. column := table.Columns[columnName]
  270. if column.Kind == PARTITION_KEY {
  271. table.PartitionKey[column.ComponentIndex] = column
  272. } else if column.Kind == CLUSTERING_KEY {
  273. table.ClusteringColumns[column.ComponentIndex] = column
  274. }
  275. }
  276. }
  277. }
  278. // returns the count of coluns with the given "kind" value.
  279. func componentColumnCountOfType(columns map[string]*ColumnMetadata, kind string) int {
  280. maxComponentIndex := -1
  281. for _, column := range columns {
  282. if column.Kind == kind && column.ComponentIndex > maxComponentIndex {
  283. maxComponentIndex = column.ComponentIndex
  284. }
  285. }
  286. return maxComponentIndex + 1
  287. }
  288. // query only for the keyspace metadata for the specified keyspace from system.schema_keyspace
  289. func getKeyspaceMetadata(
  290. session *Session,
  291. keyspaceName string,
  292. ) (*KeyspaceMetadata, error) {
  293. query := session.Query(
  294. `
  295. SELECT durable_writes, strategy_class, strategy_options
  296. FROM system.schema_keyspaces
  297. WHERE keyspace_name = ?
  298. `,
  299. keyspaceName,
  300. )
  301. // Set a routing key to avoid GetRoutingKey from computing the routing key
  302. // TODO use a separate connection (pool) for system keyspace queries.
  303. query.RoutingKey([]byte{})
  304. keyspace := &KeyspaceMetadata{Name: keyspaceName}
  305. var strategyOptionsJSON []byte
  306. err := query.Scan(
  307. &keyspace.DurableWrites,
  308. &keyspace.StrategyClass,
  309. &strategyOptionsJSON,
  310. )
  311. if err != nil {
  312. return nil, fmt.Errorf("Error querying keyspace schema: %v", err)
  313. }
  314. err = json.Unmarshal(strategyOptionsJSON, &keyspace.StrategyOptions)
  315. if err != nil {
  316. return nil, fmt.Errorf(
  317. "Invalid JSON value '%s' as strategy_options for in keyspace '%s': %v",
  318. strategyOptionsJSON, keyspace.Name, err,
  319. )
  320. }
  321. return keyspace, nil
  322. }
  323. // query for only the table metadata in the specified keyspace from system.schema_columnfamilies
  324. func getTableMetadata(
  325. session *Session,
  326. keyspaceName string,
  327. ) ([]TableMetadata, error) {
  328. query := session.Query(
  329. `
  330. SELECT
  331. columnfamily_name,
  332. key_validator,
  333. comparator,
  334. default_validator,
  335. key_aliases,
  336. column_aliases,
  337. value_alias
  338. FROM system.schema_columnfamilies
  339. WHERE keyspace_name = ?
  340. `,
  341. keyspaceName,
  342. )
  343. // Set a routing key to avoid GetRoutingKey from computing the routing key
  344. // TODO use a separate connection (pool) for system keyspace queries.
  345. query.RoutingKey([]byte{})
  346. iter := query.Iter()
  347. tables := []TableMetadata{}
  348. table := TableMetadata{Keyspace: keyspaceName}
  349. var keyAliasesJSON []byte
  350. var columnAliasesJSON []byte
  351. for iter.Scan(
  352. &table.Name,
  353. &table.KeyValidator,
  354. &table.Comparator,
  355. &table.DefaultValidator,
  356. &keyAliasesJSON,
  357. &columnAliasesJSON,
  358. &table.ValueAlias,
  359. ) {
  360. var err error
  361. // decode the key aliases
  362. if keyAliasesJSON != nil {
  363. table.KeyAliases = []string{}
  364. err = json.Unmarshal(keyAliasesJSON, &table.KeyAliases)
  365. if err != nil {
  366. iter.Close()
  367. return nil, fmt.Errorf(
  368. "Invalid JSON value '%s' as key_aliases for in table '%s': %v",
  369. keyAliasesJSON, table.Name, err,
  370. )
  371. }
  372. }
  373. // decode the column aliases
  374. if columnAliasesJSON != nil {
  375. table.ColumnAliases = []string{}
  376. err = json.Unmarshal(columnAliasesJSON, &table.ColumnAliases)
  377. if err != nil {
  378. iter.Close()
  379. return nil, fmt.Errorf(
  380. "Invalid JSON value '%s' as column_aliases for in table '%s': %v",
  381. columnAliasesJSON, table.Name, err,
  382. )
  383. }
  384. }
  385. tables = append(tables, table)
  386. table = TableMetadata{Keyspace: keyspaceName}
  387. }
  388. err := iter.Close()
  389. if err != nil && err != ErrNotFound {
  390. return nil, fmt.Errorf("Error querying table schema: %v", err)
  391. }
  392. return tables, nil
  393. }
  394. // query for only the column metadata in the specified keyspace from system.schema_columns
  395. func getColumnMetadata(
  396. session *Session,
  397. keyspaceName string,
  398. ) ([]ColumnMetadata, error) {
  399. // Deal with differences in protocol versions
  400. var stmt string
  401. var scan func(*Iter, *ColumnMetadata, *[]byte) bool
  402. if session.cfg.ProtoVersion == 1 {
  403. // V1 does not support the type column, and all returned rows are
  404. // of kind "regular".
  405. stmt = `
  406. SELECT
  407. columnfamily_name,
  408. column_name,
  409. component_index,
  410. validator,
  411. index_name,
  412. index_type,
  413. index_options
  414. FROM system.schema_columns
  415. WHERE keyspace_name = ?
  416. `
  417. scan = func(
  418. iter *Iter,
  419. column *ColumnMetadata,
  420. indexOptionsJSON *[]byte,
  421. ) bool {
  422. // all columns returned by V1 are regular
  423. column.Kind = REGULAR
  424. return iter.Scan(
  425. &column.Table,
  426. &column.Name,
  427. &column.ComponentIndex,
  428. &column.Validator,
  429. &column.Index.Name,
  430. &column.Index.Type,
  431. &indexOptionsJSON,
  432. )
  433. }
  434. } else {
  435. // V2+ supports the type column
  436. stmt = `
  437. SELECT
  438. columnfamily_name,
  439. column_name,
  440. component_index,
  441. validator,
  442. index_name,
  443. index_type,
  444. index_options,
  445. type
  446. FROM system.schema_columns
  447. WHERE keyspace_name = ?
  448. `
  449. scan = func(
  450. iter *Iter,
  451. column *ColumnMetadata,
  452. indexOptionsJSON *[]byte,
  453. ) bool {
  454. return iter.Scan(
  455. &column.Table,
  456. &column.Name,
  457. &column.ComponentIndex,
  458. &column.Validator,
  459. &column.Index.Name,
  460. &column.Index.Type,
  461. &indexOptionsJSON,
  462. &column.Kind,
  463. )
  464. }
  465. }
  466. // get the columns metadata
  467. columns := []ColumnMetadata{}
  468. column := ColumnMetadata{Keyspace: keyspaceName}
  469. var indexOptionsJSON []byte
  470. query := session.Query(stmt, keyspaceName)
  471. // Set a routing key to avoid GetRoutingKey from computing the routing key
  472. // TODO use a separate connection (pool) for system keyspace queries.
  473. query.RoutingKey([]byte{})
  474. iter := query.Iter()
  475. for scan(iter, &column, &indexOptionsJSON) {
  476. var err error
  477. // decode the index options
  478. if indexOptionsJSON != nil {
  479. err = json.Unmarshal(indexOptionsJSON, &column.Index.Options)
  480. if err != nil {
  481. iter.Close()
  482. return nil, fmt.Errorf(
  483. "Invalid JSON value '%s' as index_options for column '%s' in table '%s': %v",
  484. indexOptionsJSON,
  485. column.Name,
  486. column.Table,
  487. err,
  488. )
  489. }
  490. }
  491. columns = append(columns, column)
  492. column = ColumnMetadata{Keyspace: keyspaceName}
  493. }
  494. err := iter.Close()
  495. if err != nil && err != ErrNotFound {
  496. return nil, fmt.Errorf("Error querying column schema: %v", err)
  497. }
  498. return columns, nil
  499. }
  500. // type definition parser state
  501. type typeParser struct {
  502. input string
  503. index int
  504. }
  505. // the type definition parser result
  506. type typeParserResult struct {
  507. isComposite bool
  508. types []TypeInfo
  509. reversed []bool
  510. collections map[string]TypeInfo
  511. }
  512. // Parse the type definition used for validator and comparator schema data
  513. func parseType(def string) typeParserResult {
  514. parser := &typeParser{input: def}
  515. return parser.parse()
  516. }
  517. const (
  518. REVERSED_TYPE = "org.apache.cassandra.db.marshal.ReversedType"
  519. COMPOSITE_TYPE = "org.apache.cassandra.db.marshal.CompositeType"
  520. COLLECTION_TYPE = "org.apache.cassandra.db.marshal.ColumnToCollectionType"
  521. LIST_TYPE = "org.apache.cassandra.db.marshal.ListType"
  522. SET_TYPE = "org.apache.cassandra.db.marshal.SetType"
  523. MAP_TYPE = "org.apache.cassandra.db.marshal.MapType"
  524. )
  525. // represents a class specification in the type def AST
  526. type typeParserClassNode struct {
  527. name string
  528. params []typeParserParamNode
  529. // this is the segment of the input string that defined this node
  530. input string
  531. }
  532. // represents a class parameter in the type def AST
  533. type typeParserParamNode struct {
  534. name *string
  535. class typeParserClassNode
  536. }
  537. func (t *typeParser) parse() typeParserResult {
  538. // parse the AST
  539. ast, ok := t.parseClassNode()
  540. if !ok {
  541. // treat this is a custom type
  542. return typeParserResult{
  543. isComposite: false,
  544. types: []TypeInfo{
  545. NativeType{
  546. typ: TypeCustom,
  547. custom: t.input,
  548. },
  549. },
  550. reversed: []bool{false},
  551. collections: nil,
  552. }
  553. }
  554. // interpret the AST
  555. if strings.HasPrefix(ast.name, COMPOSITE_TYPE) {
  556. count := len(ast.params)
  557. // look for a collections param
  558. last := ast.params[count-1]
  559. collections := map[string]TypeInfo{}
  560. if strings.HasPrefix(last.class.name, COLLECTION_TYPE) {
  561. count--
  562. for _, param := range last.class.params {
  563. // decode the name
  564. var name string
  565. decoded, err := hex.DecodeString(*param.name)
  566. if err != nil {
  567. log.Printf(
  568. "Error parsing type '%s', contains collection name '%s' with an invalid format: %v",
  569. t.input,
  570. *param.name,
  571. err,
  572. )
  573. // just use the provided name
  574. name = *param.name
  575. } else {
  576. name = string(decoded)
  577. }
  578. collections[name] = param.class.asTypeInfo()
  579. }
  580. }
  581. types := make([]TypeInfo, count)
  582. reversed := make([]bool, count)
  583. for i, param := range ast.params[:count] {
  584. class := param.class
  585. reversed[i] = strings.HasPrefix(class.name, REVERSED_TYPE)
  586. if reversed[i] {
  587. class = class.params[0].class
  588. }
  589. types[i] = class.asTypeInfo()
  590. }
  591. return typeParserResult{
  592. isComposite: true,
  593. types: types,
  594. reversed: reversed,
  595. collections: collections,
  596. }
  597. } else {
  598. // not composite, so one type
  599. class := *ast
  600. reversed := strings.HasPrefix(class.name, REVERSED_TYPE)
  601. if reversed {
  602. class = class.params[0].class
  603. }
  604. typeInfo := class.asTypeInfo()
  605. return typeParserResult{
  606. isComposite: false,
  607. types: []TypeInfo{typeInfo},
  608. reversed: []bool{reversed},
  609. }
  610. }
  611. }
  612. func (class *typeParserClassNode) asTypeInfo() TypeInfo {
  613. if strings.HasPrefix(class.name, LIST_TYPE) {
  614. elem := class.params[0].class.asTypeInfo()
  615. return CollectionType{
  616. NativeType: NativeType{
  617. typ: TypeList,
  618. },
  619. Elem: elem,
  620. }
  621. }
  622. if strings.HasPrefix(class.name, SET_TYPE) {
  623. elem := class.params[0].class.asTypeInfo()
  624. return CollectionType{
  625. NativeType: NativeType{
  626. typ: TypeSet,
  627. },
  628. Elem: elem,
  629. }
  630. }
  631. if strings.HasPrefix(class.name, MAP_TYPE) {
  632. key := class.params[0].class.asTypeInfo()
  633. elem := class.params[1].class.asTypeInfo()
  634. return CollectionType{
  635. NativeType: NativeType{
  636. typ: TypeMap,
  637. },
  638. Key: key,
  639. Elem: elem,
  640. }
  641. }
  642. // must be a simple type or custom type
  643. info := NativeType{typ: getApacheCassandraType(class.name)}
  644. if info.typ == TypeCustom {
  645. // add the entire class definition
  646. info.custom = class.input
  647. }
  648. return info
  649. }
  650. // CLASS := ID [ PARAMS ]
  651. func (t *typeParser) parseClassNode() (node *typeParserClassNode, ok bool) {
  652. t.skipWhitespace()
  653. startIndex := t.index
  654. name, ok := t.nextIdentifier()
  655. if !ok {
  656. return nil, false
  657. }
  658. params, ok := t.parseParamNodes()
  659. if !ok {
  660. return nil, false
  661. }
  662. endIndex := t.index
  663. node = &typeParserClassNode{
  664. name: name,
  665. params: params,
  666. input: t.input[startIndex:endIndex],
  667. }
  668. return node, true
  669. }
  670. // PARAMS := "(" PARAM { "," PARAM } ")"
  671. // PARAM := [ PARAM_NAME ":" ] CLASS
  672. // PARAM_NAME := ID
  673. func (t *typeParser) parseParamNodes() (params []typeParserParamNode, ok bool) {
  674. t.skipWhitespace()
  675. // the params are optional
  676. if t.index == len(t.input) || t.input[t.index] != '(' {
  677. return nil, true
  678. }
  679. params = []typeParserParamNode{}
  680. // consume the '('
  681. t.index++
  682. t.skipWhitespace()
  683. for t.input[t.index] != ')' {
  684. // look for a named param, but if no colon, then we want to backup
  685. backupIndex := t.index
  686. // name will be a hex encoded version of a utf-8 string
  687. name, ok := t.nextIdentifier()
  688. if !ok {
  689. return nil, false
  690. }
  691. hasName := true
  692. // TODO handle '=>' used for DynamicCompositeType
  693. t.skipWhitespace()
  694. if t.input[t.index] == ':' {
  695. // there is a name for this parameter
  696. // consume the ':'
  697. t.index++
  698. t.skipWhitespace()
  699. } else {
  700. // no name, backup
  701. hasName = false
  702. t.index = backupIndex
  703. }
  704. // parse the next full parameter
  705. classNode, ok := t.parseClassNode()
  706. if !ok {
  707. return nil, false
  708. }
  709. if hasName {
  710. params = append(
  711. params,
  712. typeParserParamNode{name: &name, class: *classNode},
  713. )
  714. } else {
  715. params = append(
  716. params,
  717. typeParserParamNode{class: *classNode},
  718. )
  719. }
  720. t.skipWhitespace()
  721. if t.input[t.index] == ',' {
  722. // consume the comma
  723. t.index++
  724. t.skipWhitespace()
  725. }
  726. }
  727. // consume the ')'
  728. t.index++
  729. return params, true
  730. }
  731. func (t *typeParser) skipWhitespace() {
  732. for t.index < len(t.input) && isWhitespaceChar(t.input[t.index]) {
  733. t.index++
  734. }
  735. }
  736. func isWhitespaceChar(c byte) bool {
  737. return c == ' ' || c == '\n' || c == '\t'
  738. }
  739. // ID := LETTER { LETTER }
  740. // LETTER := "0"..."9" | "a"..."z" | "A"..."Z" | "-" | "+" | "." | "_" | "&"
  741. func (t *typeParser) nextIdentifier() (id string, found bool) {
  742. startIndex := t.index
  743. for t.index < len(t.input) && isIdentifierChar(t.input[t.index]) {
  744. t.index++
  745. }
  746. if startIndex == t.index {
  747. return "", false
  748. }
  749. return t.input[startIndex:t.index], true
  750. }
  751. func isIdentifierChar(c byte) bool {
  752. return (c >= '0' && c <= '9') ||
  753. (c >= 'a' && c <= 'z') ||
  754. (c >= 'A' && c <= 'Z') ||
  755. c == '-' ||
  756. c == '+' ||
  757. c == '.' ||
  758. c == '_' ||
  759. c == '&'
  760. }