joao-r-reis commented on code in PR #1822:
URL: 
https://github.com/apache/cassandra-gocql-driver/pull/1822#discussion_r1820655641


##########
conn.go:
##########
@@ -1430,13 +1612,36 @@ func (c *Conn) executeQuery(ctx context.Context, qry 
*Query) *Iter {
        case *resultVoidFrame:
                return &Iter{framer: framer}
        case *resultRowsFrame:
+               if x.meta.newMetadataID != nil {
+                       // If a RESULT/Rows message reports
+                       //      changed resultset metadata with the 
Metadata_changed flag, the reported new
+                       //      resultset metadata must be used in subsequent 
executions
+                       stmtCacheKey := 
c.session.stmtsLRU.keyFor(c.host.HostID(), c.currentKeyspace, qry.stmt)
+                       oldInflight, ok := c.session.stmtsLRU.get(stmtCacheKey)
+                       if ok {
+                               newInflight := &inflightPrepare{
+                                       done: make(chan struct{}),
+                                       preparedStatment: &preparedStatment{
+                                               id:               
oldInflight.preparedStatment.id,
+                                               resultMetadataID: 
x.meta.newMetadataID,
+                                               request:          
oldInflight.preparedStatment.request,
+                                               response:         x.meta,
+                                       },
+                               }
+                               c.session.stmtsLRU.add(stmtCacheKey, 
newInflight)

Review Comment:
   Update the `info` local variable here just in case so we ensure that the 
code in this function is looking at the updated version of the prepared 
statement.



##########
conn.go:
##########
@@ -1430,13 +1612,36 @@ func (c *Conn) executeQuery(ctx context.Context, qry 
*Query) *Iter {
        case *resultVoidFrame:
                return &Iter{framer: framer}
        case *resultRowsFrame:
+               if x.meta.newMetadataID != nil {
+                       // If a RESULT/Rows message reports
+                       //      changed resultset metadata with the 
Metadata_changed flag, the reported new
+                       //      resultset metadata must be used in subsequent 
executions
+                       stmtCacheKey := 
c.session.stmtsLRU.keyFor(c.host.HostID(), c.currentKeyspace, qry.stmt)
+                       oldInflight, ok := c.session.stmtsLRU.get(stmtCacheKey)
+                       if ok {
+                               newInflight := &inflightPrepare{
+                                       done: make(chan struct{}),
+                                       preparedStatment: &preparedStatment{
+                                               id:               
oldInflight.preparedStatment.id,
+                                               resultMetadataID: 
x.meta.newMetadataID,
+                                               request:          
oldInflight.preparedStatment.request,
+                                               response:         x.meta,
+                                       },
+                               }
+                               c.session.stmtsLRU.add(stmtCacheKey, 
newInflight)
+                               // The driver should close this done to avoid 
deadlocks of
+                               // other subsequent requests
+                               close(newInflight.done)
+                       }
+               }
+
                iter := &Iter{
                        meta:    x.meta,
                        framer:  framer,
                        numRows: x.numRows,
                }
 
-               if params.skipMeta {
+               if params.skipMeta && x.meta.noMetaData() {

Review Comment:
   I'd remove `params.skipMeta` from this check completely. If response has 
metadata we use it, if it doesn't have it then we use the metadata from the 
cache. The request flag doesn't even matter



##########
conn.go:
##########
@@ -378,10 +379,19 @@ func (s *startupCoordinator) setupConn(ctx 
context.Context) error {
        }
        defer cancel()
 
+       // Only for proto v5+.
+       // Indicates if STARTUP has been completed.
+       // github.com/apache/cassandra/blob/trunk/doc/native_protocol_v5.spec
+       // 2.3.1 Initial Handshake
+       //      In order to support both v5 and earlier formats, the v5 framing 
format is not
+       //  applied to message exchanges before an initial handshake is 
completed.
+       startupCompleted := &atomic.Bool{}

Review Comment:
   I'm going to keep this one open until we have a second reviewer to look at 
this.



##########
conn.go:
##########
@@ -1430,13 +1612,36 @@ func (c *Conn) executeQuery(ctx context.Context, qry 
*Query) *Iter {
        case *resultVoidFrame:
                return &Iter{framer: framer}
        case *resultRowsFrame:
+               if x.meta.newMetadataID != nil {
+                       // If a RESULT/Rows message reports
+                       //      changed resultset metadata with the 
Metadata_changed flag, the reported new
+                       //      resultset metadata must be used in subsequent 
executions
+                       stmtCacheKey := 
c.session.stmtsLRU.keyFor(c.host.HostID(), c.currentKeyspace, qry.stmt)
+                       oldInflight, ok := c.session.stmtsLRU.get(stmtCacheKey)
+                       if ok {
+                               newInflight := &inflightPrepare{
+                                       done: make(chan struct{}),
+                                       preparedStatment: &preparedStatment{
+                                               id:               
oldInflight.preparedStatment.id,
+                                               resultMetadataID: 
x.meta.newMetadataID,
+                                               request:          
oldInflight.preparedStatment.request,
+                                               response:         x.meta,
+                                       },
+                               }
+                               c.session.stmtsLRU.add(stmtCacheKey, 
newInflight)
+                               // The driver should close this done to avoid 
deadlocks of
+                               // other subsequent requests
+                               close(newInflight.done)

Review Comment:
   close it before even adding it to the cache, could potentially reduce 
blocking if another goroutine gets this stmt from the cache before the channel 
is closed



##########
conn.go:
##########
@@ -1430,6 +1529,34 @@ func (c *Conn) executeQuery(ctx context.Context, qry 
*Query) *Iter {
        case *resultVoidFrame:
                return &Iter{framer: framer}
        case *resultRowsFrame:
+               if x.meta.newMetadataID != nil {
+                       // Updating the result metadata id in prepared stmt
+                       //
+                       // If a RESULT/Rows message reports
+                       //      changed resultset metadata with the 
Metadata_changed flag, the reported new
+                       //      resultset metadata must be used in subsequent 
executions
+
+                       stmtCacheKey := 
c.session.stmtsLRU.keyFor(c.host.HostID(), c.currentKeyspace, qry.stmt)
+                       oldInflight, ok := c.session.stmtsLRU.get(stmtCacheKey)
+                       if !ok {
+                               // We didn't find the stmt in the cache, so we 
just re-prepare it
+                               return c.executeQuery(ctx, qry)
+                       }
+
+                       newInflight := &inflightPrepare{
+                               done: make(chan struct{}),
+                               preparedStatment: &preparedStatment{
+                                       id:               
oldInflight.preparedStatment.id,
+                                       resultMetadataID: x.meta.newMetadataID,
+                                       request:          
oldInflight.preparedStatment.request,
+                                       response:         x.meta,
+                               },
+                       }
+
+                       c.session.stmtsLRU.add(stmtCacheKey, newInflight)
+                       return c.executeQuery(ctx, qry)

Review Comment:
   The test isn't triggering the METADATA_CHANGED flag atm because of what I 
mentioned before but here's a way that it can be changed to trigger it: 
https://github.com/joao-r-reis/gocql/commit/bf39830147873c52e825c10f3855538778f3b92f
 
   
   One thing that is missing from the test though is checking result data 
integrity after the metadata change. Maybe inserting a row after the ALTER with 
data on both columns and then asserting that the data retrieved matches the 
values we just inserted.



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: [email protected]

For queries about this service, please contact Infrastructure at:
[email protected]


---------------------------------------------------------------------
To unsubscribe, e-mail: [email protected]
For additional commands, e-mail: [email protected]

Reply via email to