From: Monty <monty(a)mariadb.org>
PURGE BINARY LOGS did not always purge binary logs. This commit fixes
some of the issues and adds notifications if a binary log cannot be
purged.
User visible changes:
- 'PURGE BINARY LOG TO log_name' and 'PURGE BINARY LOGS BEFORE date'
worked differently. 'TO' ignored 'slave_connections_needed_for_purge'
while 'BEFORE' did not. Now both versions ignores the
'slave_connections_needed_for_purge variable'.
- 'PURGE BINARY LOG..' commands now returns 'note' if a binary log cannot
be deleted like
Note 1375 Binary log 'master-bin.000004' is not purged because it is
the current active binlog
- Automatic binary log purges, based on date or size, will write a
note to the error log if a binary log matching the size or date
cannot yet be deleted.
- If 'slave_connections_needed_for_purge' is set from a config or
command line, it is set to 0 if Galera is enabled and 1 otherwise
(old default). This ensures that automatic binary log purge works
with Galera as before the addition of
'slave_connections_needed_for_purge'.
If the variable is changed to 0, a warning will be printed to the error
log.
Code changes:
- Added THD argument to several purge_logs related functions that needed
THD.
- Added 'interactive' options to purge_logs functions. This allowed
me to remove testing of sql_command == SQLCOM_PURGE.
- Changed purge_logs_before_date() to first check if log is applicable
before calling can_purge_logs(). This ensures we do not get a
notification for logs that does not match the remove criteria.
- MYSQL_BIN_LOG::can_purge_log() will write notifications to the user
or error log if a log cannot yet be removed.
- log_in_use() will return reason why a binary log cannot be removed.
- Moved checking of binlog_format for Galera to be after Galera is
initialized (The old check never worked). If Galera is enabled
we now change the binlog_format to ROW, with a warning, instead of
aborting the server. If this change happens, the binlog_format variable
will be marked with AUTO or FORCED, for information_schema.system_variables,
and a warning will be printed to the error log.
- Print also a warning if FLASHBACK changes the binlog_format to ROW.
Before this was done silently.
---
mysql-test/main/mysqld--help.result | 1 +
.../binlog_flush_binlogs_delete_domain.result | 3 +
mysql-test/suite/binlog/r/binlog_index.result | 1 +
.../suite/binlog/r/binlog_xa_recover.result | 2 +
.../t/binlog_flush_binlogs_delete_domain.test | 2 +
.../binlog_encryption/binlog_index.result | 1 +
.../binlog_xa_recover.result | 2 +
.../oracle/r/binlog_ptr_mysqlbinlog.result | 1 +
.../oracle/t/binlog_ptr_mysqlbinlog.test | 2 +
mysql-test/suite/galera/r/basic.result | 6 ++
mysql-test/suite/galera/t/basic.test | 3 +
mysql-test/suite/rpl/r/purge_binlog.result | 52 ++++++++++
mysql-test/suite/rpl/r/rpl_rotate_logs.result | 2 +
mysql-test/suite/rpl/t/purge_binlog.test | 54 +++++++++++
.../r/sysvars_server_notembedded.result | 2 +-
sql/log.cc | 95 ++++++++++++++-----
sql/log.h | 8 +-
sql/mysqld.cc | 48 +++++++---
sql/mysqld.h | 1 +
sql/set_var.cc | 3 +-
sql/set_var.h | 3 +-
sql/sql_repl.cc | 26 +++--
sql/sql_repl.h | 2 +-
sql/sys_vars.cc | 21 +++-
sql/sys_vars.h | 17 ++++
25 files changed, 303 insertions(+), 55 deletions(-)
create mode 100644 mysql-test/suite/rpl/r/purge_binlog.result
create mode 100644 mysql-test/suite/rpl/t/purge_binlog.test
create mode 100644 sql/sys_vars.h
diff --git a/mysql-test/main/mysqld--help.result b/mysql-test/main/mysqld--help.result
index ee75f586970..026f135ebd5 100644
--- a/mysql-test/main/mysqld--help.result
+++ b/mysql-test/main/mysqld--help.result
@@ -1308,6 +1308,7 @@ The following specify which files/extra groups are read (specified before remain
Minimum number of connected slaves required for automatic
binary log purge with max_binlog_total_size,
binlog_expire_logs_seconds or binlog_expire_logs_days.
+ Default is 0 when Galera is enabled and 1 otherwise.
--slave-ddl-exec-mode=name
How replication events should be executed. Legal values
are STRICT and IDEMPOTENT (default). In IDEMPOTENT mode,
diff --git a/mysql-test/suite/binlog/r/binlog_flush_binlogs_delete_domain.result b/mysql-test/suite/binlog/r/binlog_flush_binlogs_delete_domain.result
index 1c11191802f..f967a367898 100644
--- a/mysql-test/suite/binlog/r/binlog_flush_binlogs_delete_domain.result
+++ b/mysql-test/suite/binlog/r/binlog_flush_binlogs_delete_domain.result
@@ -49,6 +49,9 @@ ERROR HY000: Could not delete gtid domain. Reason: binlog files may contain gtid
MDEV-31140: Missing error from DELETE_DOMAIN_ID when gtid_binlog_state partially matches GTID_LIST.
FLUSH BINARY LOGS;
PURGE BINARY LOGS TO 'master-bin.000005';
+SHOW BINARY LOGS;
+Log_name File_size
+master-bin.000005 #
SET @@SESSION.gtid_domain_id=8;
SET @@SESSION.server_id=10*8 + 1;
INSERT INTO t SELECT 1+MAX(a) FROM t;
diff --git a/mysql-test/suite/binlog/r/binlog_index.result b/mysql-test/suite/binlog/r/binlog_index.result
index 9dfda71f9a7..2d2363a7fec 100644
--- a/mysql-test/suite/binlog/r/binlog_index.result
+++ b/mysql-test/suite/binlog/r/binlog_index.result
@@ -30,6 +30,7 @@ flush logs;
flush logs;
*** must be a warning master-bin.000001 was not found ***
Warnings:
+Note 1375 Binary log 'master-bin.000004' is not purged because it is the current active binlog
Warning 1612 Being purged log master-bin.000001 was not found
*** must show one record, of the active binlog, left in the index file after PURGE ***
show binary logs;
diff --git a/mysql-test/suite/binlog/r/binlog_xa_recover.result b/mysql-test/suite/binlog/r/binlog_xa_recover.result
index f5060fd5160..da8dff2dfd1 100644
--- a/mysql-test/suite/binlog/r/binlog_xa_recover.result
+++ b/mysql-test/suite/binlog/r/binlog_xa_recover.result
@@ -89,6 +89,8 @@ master-bin.000006 # Format_desc # # SERVER_VERSION, BINLOG_VERSION
master-bin.000006 # Gtid_list # # [#-#-#]
master-bin.000006 # Binlog_checkpoint # # master-bin.000004
PURGE BINARY LOGS TO "master-bin.000006";
+Warnings:
+Note 1375 Binary log 'master-bin.000004' is not purged because it is in use by an active XID transaction
show binary logs;
Log_name File_size
master-bin.000004 #
diff --git a/mysql-test/suite/binlog/t/binlog_flush_binlogs_delete_domain.test b/mysql-test/suite/binlog/t/binlog_flush_binlogs_delete_domain.test
index 1643ecff72d..4a974c642c0 100644
--- a/mysql-test/suite/binlog/t/binlog_flush_binlogs_delete_domain.test
+++ b/mysql-test/suite/binlog/t/binlog_flush_binlogs_delete_domain.test
@@ -91,6 +91,8 @@ while ($domain_cnt)
FLUSH BINARY LOGS;
--let $purge_to_binlog= query_get_value(SHOW MASTER STATUS, File, 1)
--eval PURGE BINARY LOGS TO '$purge_to_binlog'
+--replace_column 2 #
+SHOW BINARY LOGS;
--eval SET @@SESSION.gtid_domain_id=$err_domain_id
--eval SET @@SESSION.server_id=10*$err_domain_id + $err_server_id
eval INSERT INTO t SELECT 1+MAX(a) FROM t;
diff --git a/mysql-test/suite/binlog_encryption/binlog_index.result b/mysql-test/suite/binlog_encryption/binlog_index.result
index 9dfda71f9a7..2d2363a7fec 100644
--- a/mysql-test/suite/binlog_encryption/binlog_index.result
+++ b/mysql-test/suite/binlog_encryption/binlog_index.result
@@ -30,6 +30,7 @@ flush logs;
flush logs;
*** must be a warning master-bin.000001 was not found ***
Warnings:
+Note 1375 Binary log 'master-bin.000004' is not purged because it is the current active binlog
Warning 1612 Being purged log master-bin.000001 was not found
*** must show one record, of the active binlog, left in the index file after PURGE ***
show binary logs;
diff --git a/mysql-test/suite/binlog_encryption/binlog_xa_recover.result b/mysql-test/suite/binlog_encryption/binlog_xa_recover.result
index 3e4ed42cf7c..7aacd0e034a 100644
--- a/mysql-test/suite/binlog_encryption/binlog_xa_recover.result
+++ b/mysql-test/suite/binlog_encryption/binlog_xa_recover.result
@@ -93,6 +93,8 @@ master-bin.000006 # Start_encryption # #
master-bin.000006 # Gtid_list # # [#-#-#]
master-bin.000006 # Binlog_checkpoint # # master-bin.000004
PURGE BINARY LOGS TO "master-bin.000006";
+Warnings:
+Note 1375 Binary log 'master-bin.000004' is not purged because it is in use by an active XID transaction
show binary logs;
Log_name File_size
master-bin.000004 #
diff --git a/mysql-test/suite/compat/oracle/r/binlog_ptr_mysqlbinlog.result b/mysql-test/suite/compat/oracle/r/binlog_ptr_mysqlbinlog.result
index 0656a685976..543ec5336d8 100644
--- a/mysql-test/suite/compat/oracle/r/binlog_ptr_mysqlbinlog.result
+++ b/mysql-test/suite/compat/oracle/r/binlog_ptr_mysqlbinlog.result
@@ -1,3 +1,4 @@
+call mtr.add_suppression("Binlog_format changed to.*flashback");
SET @@SQL_MODE = 'ORACLE';
##########################################################################
# Test verifies Gtid_log_event/Xid_log_event specific print #
diff --git a/mysql-test/suite/compat/oracle/t/binlog_ptr_mysqlbinlog.test b/mysql-test/suite/compat/oracle/t/binlog_ptr_mysqlbinlog.test
index bda32af5d4e..165f2cc0afe 100644
--- a/mysql-test/suite/compat/oracle/t/binlog_ptr_mysqlbinlog.test
+++ b/mysql-test/suite/compat/oracle/t/binlog_ptr_mysqlbinlog.test
@@ -18,6 +18,8 @@
--source include/have_log_bin.inc
--source include/have_innodb.inc
+call mtr.add_suppression("Binlog_format changed to.*flashback");
+
let $MYSQLD_DATADIR= `select @@datadir`;
SET @@SQL_MODE = 'ORACLE';
diff --git a/mysql-test/suite/galera/r/basic.result b/mysql-test/suite/galera/r/basic.result
index 10f180e7a94..7b4cbd93c7b 100644
--- a/mysql-test/suite/galera/r/basic.result
+++ b/mysql-test/suite/galera/r/basic.result
@@ -1,5 +1,11 @@
connection node_2;
connection node_1;
+select @@slave_connections_needed_for_purge;
+@@slave_connections_needed_for_purge
+0
+select VARIABLE_NAME, GLOBAL_VALUE, GLOBAL_VALUE_ORIGIN from information_schema.system_variables where variable_name="slave_connections_needed_for_purge";
+VARIABLE_NAME GLOBAL_VALUE GLOBAL_VALUE_ORIGIN
+SLAVE_CONNECTIONS_NEEDED_FOR_PURGE 0 AUTO
USE test;
CREATE TABLE t1(c1 INT PRIMARY KEY) ENGINE=INNODB;
INSERT INTO t1 VALUES (1), (2), (3), (4), (5);
diff --git a/mysql-test/suite/galera/t/basic.test b/mysql-test/suite/galera/t/basic.test
index 8fc6eee3b3b..a70ee962bc0 100644
--- a/mysql-test/suite/galera/t/basic.test
+++ b/mysql-test/suite/galera/t/basic.test
@@ -1,6 +1,9 @@
--source include/galera_cluster.inc
--source include/have_innodb.inc
+select @@slave_connections_needed_for_purge;
+select VARIABLE_NAME, GLOBAL_VALUE, GLOBAL_VALUE_ORIGIN from information_schema.system_variables where variable_name="slave_connections_needed_for_purge";
+
USE test;
CREATE TABLE t1(c1 INT PRIMARY KEY) ENGINE=INNODB;
INSERT INTO t1 VALUES (1), (2), (3), (4), (5);
diff --git a/mysql-test/suite/rpl/r/purge_binlog.result b/mysql-test/suite/rpl/r/purge_binlog.result
new file mode 100644
index 00000000000..6be060e5774
--- /dev/null
+++ b/mysql-test/suite/rpl/r/purge_binlog.result
@@ -0,0 +1,52 @@
+include/master-slave.inc
+[connection master]
+#
+# MDEV-34504 PURGE BINARY LOGS not working anymore
+#
+select @@slave_connections_needed_for_purge;
+@@slave_connections_needed_for_purge
+0
+set @old_dbug= @@global.debug_dbug;
+create table t1 (a int, b varchar(32768));
+insert into t1 values(1,repeat("a",32768));
+connection slave;
+select a from t1;
+a
+1
+set @@global.debug_dbug= "+d,pause_before_io_read_event";
+connection master;
+insert into t1 values(2,repeat("b",32768));
+insert into t1 values(3,repeat("c",32768));
+connection slave;
+set debug_sync='now wait_for io_thread_at_read_event';
+select a from t1;
+a
+1
+connection master;
+FLUSH BINARY LOGS;
+SHOW BINARY LOGS;
+Log_name File_size
+master-bin.000001 #
+master-bin.000002 #
+PURGE BINARY LOGS TO 'master-bin.000002';
+Warnings:
+Note 1375 Binary log XXX is not purged because it is in use by a slave thread
+SHOW BINARY LOGS;
+Log_name File_size
+master-bin.000001 #
+master-bin.000002 #
+connection slave;
+set @@global.debug_dbug= @old_dbug;
+set debug_sync='now signal io_thread_continue_read_event';
+connection master;
+connection slave;
+select count(*) from t1;
+count(*)
+103
+connection master;
+PURGE BINARY LOGS TO 'master-bin.000002';
+SHOW BINARY LOGS;
+Log_name File_size
+master-bin.000002 #
+drop table t1;
+include/rpl_end.inc
diff --git a/mysql-test/suite/rpl/r/rpl_rotate_logs.result b/mysql-test/suite/rpl/r/rpl_rotate_logs.result
index 4b7c1642a11..8e8f0026c93 100644
--- a/mysql-test/suite/rpl/r/rpl_rotate_logs.result
+++ b/mysql-test/suite/rpl/r/rpl_rotate_logs.result
@@ -70,6 +70,8 @@ master-bin.000002 #
master-bin.000003 #
SELECT @time_for_purge:=DATE_ADD('tmpval', INTERVAL 1 SECOND);
purge master logs before (@time_for_purge);
+Warnings:
+Note 1375 Binary log 'master-bin.000003' is not purged because it is the current active binlog
show binary logs;
Log_name File_size
master-bin.000003 #
diff --git a/mysql-test/suite/rpl/t/purge_binlog.test b/mysql-test/suite/rpl/t/purge_binlog.test
new file mode 100644
index 00000000000..5c804ca22bb
--- /dev/null
+++ b/mysql-test/suite/rpl/t/purge_binlog.test
@@ -0,0 +1,54 @@
+--source include/have_debug.inc
+--source include/have_debug_sync.inc
+--source include/master-slave.inc
+--source include/have_binlog_format_row.inc
+
+--echo #
+--echo # MDEV-34504 PURGE BINARY LOGS not working anymore
+--echo #
+
+select @@slave_connections_needed_for_purge;
+set @old_dbug= @@global.debug_dbug;
+
+create table t1 (a int, b varchar(32768));
+insert into t1 values(1,repeat("a",32768));
+--sync_slave_with_master
+select a from t1;
+set @@global.debug_dbug= "+d,pause_before_io_read_event";
+--connection master
+insert into t1 values(2,repeat("b",32768));
+insert into t1 values(3,repeat("c",32768));
+--connection slave
+set debug_sync='now wait_for io_thread_at_read_event';
+select a from t1;
+--connection master
+--disable_query_log
+let $i=100;
+while ($i)
+{
+--eval insert into t1 values($i+4,repeat(char(64+$i),32768));
+--dec $i
+}
+--enable_query_log
+
+FLUSH BINARY LOGS;
+--replace_column 2 #
+SHOW BINARY LOGS;
+--let $purge_to_binlog= query_get_value(SHOW MASTER STATUS, File, 1)
+--replace_regex /Binary log.*is not/Binary log XXX is not/
+--eval PURGE BINARY LOGS TO '$purge_to_binlog'
+--replace_column 2 #
+SHOW BINARY LOGS;
+--connection slave
+set @@global.debug_dbug= @old_dbug;
+set debug_sync='now signal io_thread_continue_read_event';
+--connection master
+--sync_slave_with_master
+select count(*) from t1;
+--connection master
+--eval PURGE BINARY LOGS TO '$purge_to_binlog'
+--replace_column 2 #
+SHOW BINARY LOGS;
+drop table t1;
+
+--source include/rpl_end.inc
diff --git a/mysql-test/suite/sys_vars/r/sysvars_server_notembedded.result b/mysql-test/suite/sys_vars/r/sysvars_server_notembedded.result
index 2740319dd05..0b2feb675b4 100644
--- a/mysql-test/suite/sys_vars/r/sysvars_server_notembedded.result
+++ b/mysql-test/suite/sys_vars/r/sysvars_server_notembedded.result
@@ -3995,7 +3995,7 @@ COMMAND_LINE_ARGUMENT OPTIONAL
VARIABLE_NAME SLAVE_CONNECTIONS_NEEDED_FOR_PURGE
VARIABLE_SCOPE GLOBAL
VARIABLE_TYPE INT UNSIGNED
-VARIABLE_COMMENT Minimum number of connected slaves required for automatic binary log purge with max_binlog_total_size, binlog_expire_logs_seconds or binlog_expire_logs_days.
+VARIABLE_COMMENT Minimum number of connected slaves required for automatic binary log purge with max_binlog_total_size, binlog_expire_logs_seconds or binlog_expire_logs_days. Default is 0 when Galera is enabled and 1 otherwise.
NUMERIC_MIN_VALUE 0
NUMERIC_MAX_VALUE 4294967295
NUMERIC_BLOCK_SIZE 1
diff --git a/sql/log.cc b/sql/log.cc
index c27c4f3353b..1384fb0b3e7 100644
--- a/sql/log.cc
+++ b/sql/log.cc
@@ -4791,8 +4791,8 @@ int MYSQL_BIN_LOG::purge_first_log(Relay_log_info* rli, bool included)
DBUG_EXECUTE_IF("crash_before_purge_logs", DBUG_SUICIDE(););
- rli->relay_log.purge_logs(to_purge_if_included, included,
- 0, 0, &log_space_reclaimed);
+ rli->relay_log.purge_logs(current_thd, to_purge_if_included, included,
+ 0, 0, 0, &log_space_reclaimed);
mysql_mutex_lock(&rli->log_space_lock);
rli->log_space_total-= log_space_reclaimed;
@@ -4859,16 +4859,17 @@ int MYSQL_BIN_LOG::update_log_index(LOG_INFO* log_info, bool need_update_threads
mysql_file_stat() or mysql_file_delete()
*/
-int MYSQL_BIN_LOG::purge_logs(const char *to_log,
+int MYSQL_BIN_LOG::purge_logs(THD *thd,
+ const char *to_log,
bool included,
bool need_mutex,
- bool need_update_threads,
+ bool need_update_threads,
+ bool interactive,
ulonglong *reclaimed_space)
{
int error= 0;
bool exit_loop= 0;
LOG_INFO log_info;
- THD *thd= current_thd;
DBUG_ENTER("purge_logs");
DBUG_PRINT("info",("to_log= %s",to_log));
@@ -4894,7 +4895,7 @@ int MYSQL_BIN_LOG::purge_logs(const char *to_log,
if (unlikely((error=find_log_pos(&log_info, NullS, 0 /*no mutex*/))))
goto err;
while ((strcmp(to_log,log_info.log_file_name) || (exit_loop=included)) &&
- can_purge_log(log_info.log_file_name))
+ can_purge_log(thd, log_info.log_file_name, interactive))
{
if (unlikely((error= register_purge_index_entry(log_info.log_file_name))))
{
@@ -4902,7 +4903,6 @@ int MYSQL_BIN_LOG::purge_logs(const char *to_log,
log_info.log_file_name);
goto err;
}
-
if (find_next_log(&log_info, 0) || exit_loop)
break;
}
@@ -5243,13 +5243,13 @@ int MYSQL_BIN_LOG::purge_index_entry(THD *thd, ulonglong *reclaimed_space,
mysql_file_stat() or mysql_file_delete()
*/
-int MYSQL_BIN_LOG::purge_logs_before_date(time_t purge_time)
+int MYSQL_BIN_LOG::purge_logs_before_date(THD *thd, time_t purge_time,
+ bool interactive)
{
int error;
char to_log[FN_REFLEN];
LOG_INFO log_info;
MY_STAT stat_area;
- THD *thd= current_thd;
DBUG_ENTER("purge_logs_before_date");
mysql_mutex_lock(&LOCK_index);
@@ -5258,7 +5258,7 @@ int MYSQL_BIN_LOG::purge_logs_before_date(time_t purge_time)
if (unlikely((error=find_log_pos(&log_info, NullS, 0 /*no mutex*/))))
goto err;
- while (can_purge_log(log_info.log_file_name))
+ for (;;)
{
if (!mysql_file_stat(m_key_file_log,
log_info.log_file_name, &stat_area, MYF(0)))
@@ -5296,7 +5296,8 @@ int MYSQL_BIN_LOG::purge_logs_before_date(time_t purge_time)
}
else
{
- if (stat_area.st_mtime >= purge_time)
+ if (stat_area.st_mtime >= purge_time ||
+ !can_purge_log(thd, log_info.log_file_name, interactive))
break;
strmake_buf(to_log, log_info.log_file_name);
}
@@ -5307,7 +5308,7 @@ int MYSQL_BIN_LOG::purge_logs_before_date(time_t purge_time)
if (to_log[0])
{
ulonglong reclaimed_space= 0;
- error= purge_logs(to_log, 1, 0, 1, &reclaimed_space);
+ error= purge_logs(thd, to_log, 1, 0, 1, interactive, &reclaimed_space);
binlog_space_total-= reclaimed_space;
}
@@ -5338,6 +5339,7 @@ int MYSQL_BIN_LOG::real_purge_logs_by_size(ulonglong binlog_pos)
MY_STAT stat_area;
char to_log[FN_REFLEN];
ulonglong found_space= 0;
+ THD *thd= current_thd;
DBUG_ENTER("real_purge_logs_by_size");
mysql_mutex_lock(&LOCK_index);
@@ -5351,7 +5353,7 @@ int MYSQL_BIN_LOG::real_purge_logs_by_size(ulonglong binlog_pos)
goto err;
to_log[0] = 0;
- while (can_purge_log(log_info.log_file_name))
+ while (can_purge_log(thd, log_info.log_file_name, 0))
{
if (!mysql_file_stat(m_key_file_log, log_info.log_file_name, &stat_area,
MYF(0)))
@@ -5394,8 +5396,9 @@ int MYSQL_BIN_LOG::real_purge_logs_by_size(ulonglong binlog_pos)
if (found_space)
{
ulonglong reclaimed_space= 0;
- purge_logs(to_log, true, false /*need_lock_index=false*/,
+ purge_logs(thd, to_log, true, false /*need_lock_index=false*/,
true /*need_update_threads=true*/,
+ false /* not interactive */,
&reclaimed_space);
DBUG_ASSERT(reclaimed_space == found_space);
binlog_space_total-= reclaimed_space;
@@ -5414,6 +5417,10 @@ int MYSQL_BIN_LOG::real_purge_logs_by_size(ulonglong binlog_pos)
}
/*
+ @param THD thd, may be null at startup
+ @param log_file_name_arg Name of log file to check
+ @param interactive True if called by a PURGE BINLOG command.
+
The following variables are here to allows us to quickly check if
the can_purge_log(log_file_name_arg) name will fail in the
'log_in_use' call.
@@ -5428,18 +5435,23 @@ int MYSQL_BIN_LOG::real_purge_logs_by_size(ulonglong binlog_pos)
static bool waiting_for_slave_to_change_binlog= 0;
static ulonglong purge_sending_new_binlog_file= 0;
static char purge_binlog_name[FN_REFLEN];
+static bool purge_warning_given= 0;
bool
-MYSQL_BIN_LOG::can_purge_log(const char *log_file_name_arg)
+MYSQL_BIN_LOG::can_purge_log(THD *thd, const char *log_file_name_arg,
+ bool interactive)
{
- THD *thd= current_thd; // May be NULL at startup
- bool res;
+ int res;
+ const char *reason;
if (is_active(log_file_name_arg) ||
(!is_relay_log && waiting_for_slave_to_change_binlog &&
purge_sending_new_binlog_file == sending_new_binlog_file &&
!strcmp(log_file_name_arg, purge_binlog_name)))
- return false;
+ {
+ reason= "it is the current active binlog";
+ goto error;
+ }
DBUG_ASSERT(!is_relay_log || binlog_xid_count_list.is_empty());
if (!is_relay_log)
@@ -5455,7 +5467,10 @@ MYSQL_BIN_LOG::can_purge_log(const char *log_file_name_arg)
}
mysql_mutex_unlock(&LOCK_xid_list);
if (b)
- return false;
+ {
+ reason= "it is in use by an active XID transaction";
+ goto error;
+ }
}
if (!is_relay_log)
@@ -5464,8 +5479,7 @@ MYSQL_BIN_LOG::can_purge_log(const char *log_file_name_arg)
purge_sending_new_binlog_file= sending_new_binlog_file;
}
if ((res= log_in_use(log_file_name_arg,
- (is_relay_log ||
- (thd && thd->lex->sql_command == SQLCOM_PURGE)) ?
+ (is_relay_log || interactive) ?
0 : slave_connections_needed_for_purge)))
{
if (!is_relay_log)
@@ -5473,9 +5487,39 @@ MYSQL_BIN_LOG::can_purge_log(const char *log_file_name_arg)
waiting_for_slave_to_change_binlog= 1;
strmake(purge_binlog_name, log_file_name_arg,
sizeof(purge_binlog_name)-1);
+ if (res == 1)
+ reason= "it is in use by a slave thread";
+ else
+ reason= "less than 'slave_connections_needed_for_purge' slaves has "
+ "processed it";
+ goto error;
}
}
- return !res;
+ /* We can purge this file, reset for next failure */
+ purge_warning_given= 0;
+ return 1;
+
+error:
+ if (!is_relay_log && (interactive || !purge_warning_given))
+ {
+ /* Remove directory (to keep things shorter and compatible */
+ log_file_name_arg+= dirname_length(log_file_name_arg);
+
+ /* purge_warning_given is reset after next sucessful purge */
+ purge_warning_given= 1;
+ if (interactive)
+ {
+ my_printf_error(ER_BINLOG_PURGE_PROHIBITED,
+ "Binary log '%s' is not purged because %s",
+ MYF(ME_NOTE), log_file_name_arg, reason);
+ }
+ else
+ {
+ sql_print_information("Binary log '%s' is not purged because %s",
+ log_file_name_arg, reason);
+ }
+ }
+ return 0;
}
#endif /* HAVE_REPLICATION */
@@ -7611,16 +7655,17 @@ void MYSQL_BIN_LOG::purge(bool all)
{
mysql_mutex_assert_not_owner(&LOCK_log);
#ifdef HAVE_REPLICATION
+ THD *thd= current_thd;
if (binlog_expire_logs_seconds)
{
- DEBUG_SYNC(current_thd, "at_purge_logs_before_date");
+ DEBUG_SYNC(thd, "at_purge_logs_before_date");
time_t purge_time= my_time(0) - binlog_expire_logs_seconds;
DBUG_EXECUTE_IF("expire_logs_always", { purge_time = my_time(0); });
if (purge_time >= 0)
{
- purge_logs_before_date(purge_time);
+ purge_logs_before_date(thd, purge_time, 0);
}
- DEBUG_SYNC(current_thd, "after_purge_logs_before_date");
+ DEBUG_SYNC(thd, "after_purge_logs_before_date");
}
if (all && binlog_space_limit)
{
diff --git a/sql/log.h b/sql/log.h
index 23513c75ac5..c8a8eb72b37 100644
--- a/sql/log.h
+++ b/sql/log.h
@@ -1035,7 +1035,7 @@ class MYSQL_BIN_LOG: public TC_LOG, private Event_log
void mark_xid_done(ulong cookie, bool write_checkpoint);
void make_log_name(char* buf, const char* log_ident);
bool is_active(const char* log_file_name);
- bool can_purge_log(const char *log_file_name);
+ bool can_purge_log(THD *thd, const char *log_file_name, bool interactive);
int update_log_index(LOG_INFO* linfo, bool need_update_threads);
int rotate(bool force_rotate, bool* check_purge);
void checkpoint_and_purge(ulong binlog_id);
@@ -1054,10 +1054,10 @@ class MYSQL_BIN_LOG: public TC_LOG, private Event_log
@retval other Failure
*/
bool flush_and_sync(bool *synced);
- int purge_logs(const char *to_log, bool included,
- bool need_mutex, bool need_update_threads,
+ int purge_logs(THD *thd, const char *to_log, bool included,
+ bool need_mutex, bool need_update_threads, bool interactive,
ulonglong *decrease_log_space);
- int purge_logs_before_date(time_t purge_time);
+ int purge_logs_before_date(THD *thd, time_t purge_time, bool interactive);
int purge_first_log(Relay_log_info* rli, bool included);
int count_binlog_space();
void count_binlog_space_with_mutex()
diff --git a/sql/mysqld.cc b/sql/mysqld.cc
index 2b12001de9e..604eaadcc92 100644
--- a/sql/mysqld.cc
+++ b/sql/mysqld.cc
@@ -52,6 +52,7 @@
#include "sql_manager.h" // stop_handle_manager, start_handle_manager
#include "sql_expression_cache.h" // subquery_cache_miss, subquery_cache_hit
#include "sys_vars_shared.h"
+#include "sys_vars.h"
#include "ddl_log.h"
#include "optimizer_defaults.h"
@@ -519,6 +520,8 @@ uint default_password_lifetime;
my_bool disconnect_on_expired_password;
bool max_user_connections_checking=0;
+bool slave_connections_needed_for_purge_option_used= 0;
+
/**
Limit of the total number of prepared statements in the server.
Is necessary to protect the server against out-of-memory attacks.
@@ -5847,7 +5850,28 @@ int mysqld_main(int argc, char **argv)
#ifdef WITH_WSREP
wsrep_set_wsrep_on(nullptr);
if (WSREP_ON && wsrep_check_opts()) unireg_abort(1);
-#endif
+
+ if (!opt_bootstrap && WSREP_PROVIDER_EXISTS && WSREP_ON)
+ {
+ if (global_system_variables.binlog_format != BINLOG_FORMAT_ROW)
+ {
+ sql_print_warning("Binlog_format changed to \"ROW\" because of Galera");
+ global_system_variables.binlog_format= BINLOG_FORMAT_ROW;
+ mark_binlog_format_used(binlog_format_used);
+ }
+ binlog_format_used= 1;
+ if (!slave_connections_needed_for_purge_option_used)
+ {
+ slave_connections_needed_for_purge=
+ internal_slave_connections_needed_for_purge= 0;
+ mark_slave_connections_needed_for_purge_as_auto();
+ sql_print_information(
+ "slave_connections_needed_for_purge changed to 0 because "
+ "of Galera. Change it to 1 or higher if this Galera node "
+ "is also Master in a normal replication setup");
+ }
+ }
+#endif /* WITH_WSREP */
#ifdef _WIN32
/*
@@ -8219,7 +8243,9 @@ mysqld_get_one_option(const struct my_option *opt, const char *argument,
((enum_slave_parallel_mode)opt_slave_parallel_mode);
break;
}
-
+ case (int) OPT_SLAVE_CONNECTIONS_NEEDED_FOR_PURGE:
+ slave_connections_needed_for_purge_option_used= 1;
+ break;
case (int)OPT_BINLOG_IGNORE_DB:
{
binlog_filter->add_ignore_db(argument);
@@ -8714,18 +8740,14 @@ static int get_options(int *argc_ptr, char ***argv_ptr)
opt_bin_log= opt_bin_log_used= 1;
/* Force format to row */
+ if (global_system_variables.binlog_format != BINLOG_FORMAT_ROW)
+ {
+ sql_print_warning("Binlog_format changed to \"ROW\" because of "
+ "flashback");
+ global_system_variables.binlog_format= BINLOG_FORMAT_ROW;
+ mark_binlog_format_used(binlog_format_used);
+ }
binlog_format_used= 1;
- global_system_variables.binlog_format= BINLOG_FORMAT_ROW;
- }
-
- if (!opt_bootstrap && WSREP_PROVIDER_EXISTS && WSREP_ON &&
- global_system_variables.binlog_format != BINLOG_FORMAT_ROW)
- {
-
- WSREP_ERROR ("Only binlog_format = 'ROW' is currently supported. "
- "Configured value: '%s'. Please adjust your configuration.",
- binlog_format_names[global_system_variables.binlog_format]);
- return 1;
}
// Synchronize @@global.autocommit on --autocommit
diff --git a/sql/mysqld.h b/sql/mysqld.h
index 33492ad7286..61313079f3f 100644
--- a/sql/mysqld.h
+++ b/sql/mysqld.h
@@ -852,6 +852,7 @@ enum options_mysqld
OPT_SILENT,
OPT_SKIP_HOST_CACHE,
OPT_SLAVE_PARALLEL_MODE,
+ OPT_SLAVE_CONNECTIONS_NEEDED_FOR_PURGE,
OPT_SSL_CA,
OPT_SSL_CAPATH,
OPT_SSL_CERT,
diff --git a/sql/set_var.cc b/sql/set_var.cc
index 61528b45de6..de537174825 100644
--- a/sql/set_var.cc
+++ b/sql/set_var.cc
@@ -1119,7 +1119,8 @@ int fill_sysvars(THD *thd, TABLE_LIST *tables, COND *cond)
{ STRING_WITH_LEN("AUTO") },
{ STRING_WITH_LEN("SQL") },
{ STRING_WITH_LEN("COMPILE-TIME") },
- { STRING_WITH_LEN("ENVIRONMENT") }
+ { STRING_WITH_LEN("ENVIRONMENT") },
+ { STRING_WITH_LEN("FORCED") },
};
const LEX_CSTRING *origin= origins + var->value_origin;
fields[3]->store(origin->str, origin->length, scs);
diff --git a/sql/set_var.h b/sql/set_var.h
index aed4955ef62..94ddef6a127 100644
--- a/sql/set_var.h
+++ b/sql/set_var.h
@@ -66,7 +66,8 @@ class sys_var: protected Value_source // for double_from_string_with_check
READONLY=1024, ALLOCATED=2048, PARSE_EARLY=4096,
NO_SET_STATEMENT=8192, AUTO_SET=16384};
enum { NO_GETOPT=-1, GETOPT_ONLY_HELP=-2 };
- enum where { CONFIG, COMMAND_LINE, AUTO, SQL, COMPILE_TIME, ENV };
+ /* If where is changed, change also GLOBAL_VALUE_ORIGIN in set_var.h */
+ enum where { CONFIG, COMMAND_LINE, AUTO, SQL, COMPILE_TIME, ENV, FORCED };
/**
Enumeration type to indicate for a system variable whether
diff --git a/sql/sql_repl.cc b/sql/sql_repl.cc
index 895ff090da6..22bc1befa6e 100644
--- a/sql/sql_repl.cc
+++ b/sql/sql_repl.cc
@@ -612,14 +612,26 @@ static my_bool log_in_use_callback(THD *thd, st_log_in_use *arg)
}
-bool log_in_use(const char* log_name, uint min_connected)
+/*
+ Check if a log is in use.
+
+ @return 0 Not used
+ @return 1 A slave is reading from the log
+ @return 2 There are less than 'min_connected' slaves that
+ has recived the log.
+*/
+
+int log_in_use(const char* log_name, uint min_connected)
{
st_log_in_use arg;
arg.log_name= log_name;
arg.connected_slaves= 0;
- return ((server_threads.iterate(log_in_use_callback, &arg) ||
- arg.connected_slaves < min_connected));
+ if (server_threads.iterate(log_in_use_callback, &arg))
+ return 1;
+ if (arg.connected_slaves < min_connected)
+ return 2;
+ return 0;
}
@@ -659,8 +671,8 @@ bool purge_master_logs(THD* thd, const char* to_log)
mysql_bin_log.make_log_name(search_file_name, to_log);
return purge_error_message(thd,
- mysql_bin_log.purge_logs(search_file_name, 0, 1,
- 1, NULL));
+ mysql_bin_log.purge_logs(thd, search_file_name,
+ 0, 1, 1, 1, NULL));
}
@@ -683,7 +695,9 @@ bool purge_master_logs_before_date(THD* thd, time_t purge_time)
return 0;
}
return purge_error_message(thd,
- mysql_bin_log.purge_logs_before_date(purge_time));
+ mysql_bin_log.purge_logs_before_date(thd,
+ purge_time,
+ 1));
}
void set_read_error(binlog_send_info *info, int error)
diff --git a/sql/sql_repl.h b/sql/sql_repl.h
index 51b6a599d5f..c03384aa5a3 100644
--- a/sql/sql_repl.h
+++ b/sql/sql_repl.h
@@ -38,7 +38,7 @@ int reset_master(THD* thd, rpl_gtid *init_state, uint32 init_state_len,
ulong next_log_number);
bool purge_master_logs(THD* thd, const char* to_log);
bool purge_master_logs_before_date(THD* thd, time_t purge_time);
-bool log_in_use(const char* log_name, uint min_connections);
+int log_in_use(const char* log_name, uint min_connections);
void adjust_linfo_offsets(my_off_t purge_offset);
void show_binlogs_get_fields(THD *thd, List<Item> *field_list);
bool show_binlogs(THD* thd);
diff --git a/sql/sys_vars.cc b/sql/sys_vars.cc
index a2705990736..35a29c29ef8 100644
--- a/sql/sys_vars.cc
+++ b/sql/sys_vars.cc
@@ -35,6 +35,7 @@
#include "sql_priv.h"
#include "sql_class.h" // set_var.h: THD
#include "sys_vars.inl"
+#include "sys_vars.h"
#include "my_sys.h"
#include "events.h"
@@ -704,6 +705,13 @@ Sys_binlog_format(
NO_MUTEX_GUARD, NOT_IN_BINLOG, ON_CHECK(binlog_format_check),
ON_UPDATE(fix_binlog_format_after_update));
+
+void mark_binlog_format_used(bool forced)
+{
+ Sys_binlog_format.value_origin= forced ? sys_var::FORCED : sys_var::AUTO;
+}
+
+
static bool binlog_direct_check(sys_var *self, THD *thd, set_var *var)
{
if (var->type == OPT_GLOBAL)
@@ -1265,7 +1273,7 @@ static bool update_binlog_space_limit(sys_var *, THD *,
mysql_bin_log.count_binlog_space();
/* Inform can_purge_log() that it should do a recheck of log_in_use() */
sending_new_binlog_file++;
- mysql_bin_log.unlock_index();
+ mysql_bin_log.unlock_index();
mysql_bin_log.purge(1);
return 0;
}
@@ -1274,6 +1282,7 @@ static bool update_binlog_space_limit(sys_var *, THD *,
return 0;
}
+
static Sys_var_on_access_global<Sys_var_ulonglong,
PRIV_SET_SYSTEM_GLOBAL_VAR_MAX_BINLOG_CACHE_SIZE>
Sys_max_binlog_total_size(
@@ -1303,13 +1312,19 @@ Sys_slave_connections_needed_for_purge(
"slave_connections_needed_for_purge",
"Minimum number of connected slaves required for automatic binary "
"log purge with max_binlog_total_size, binlog_expire_logs_seconds "
- "or binlog_expire_logs_days.",
+ "or binlog_expire_logs_days. Default is 0 when Galera is enabled and 1 "
+ "otherwise.",
GLOBAL_VAR(internal_slave_connections_needed_for_purge),
- CMD_LINE(REQUIRED_ARG),
+ CMD_LINE(REQUIRED_ARG, OPT_SLAVE_CONNECTIONS_NEEDED_FOR_PURGE),
VALID_RANGE(0, UINT_MAX), DEFAULT(1), BLOCK_SIZE(1),
NO_MUTEX_GUARD, NOT_IN_BINLOG, ON_CHECK(0),
ON_UPDATE(update_binlog_space_limit));
+void mark_slave_connections_needed_for_purge_as_auto()
+{
+ Sys_slave_connections_needed_for_purge.value_origin= sys_var::AUTO;
+}
+
static Sys_var_mybool Sys_flush(
"flush", "Flush MyISAM tables to disk between SQL commands",
GLOBAL_VAR(myisam_flush),
diff --git a/sql/sys_vars.h b/sql/sys_vars.h
new file mode 100644
index 00000000000..8f4eac38cd0
--- /dev/null
+++ b/sql/sys_vars.h
@@ -0,0 +1,17 @@
+/* Copyright (c) 2024, MariaDB Corporation.
+
+ This program is free software; you can redistribute it and/or modify
+ it under the terms of the GNU General Public License as published by
+ the Free Software Foundation; version 2 of the License.
+
+ This program is distributed in the hope that it will be useful,
+ but WITHOUT ANY WARRANTY; without even the implied warranty of
+ MERCHANTABILITY or FITNESS FOR A PARTICULAR PURPOSE. See the
+ GNU General Public License for more details.
+
+ You should have received a copy of the GNU General Public License
+ along with this program; if not, write to the Free Software
+ Foundation, Inc., 51 Franklin St, Fifth Floor, Boston, MA 02110-1335 USA */
+
+extern void mark_binlog_format_used(bool forced);
+extern void mark_slave_connections_needed_for_purge_as_auto();
--
2.30.2
1
0
From: poempeng <poempeng(a)tencent.com>
During the commit stage of a large transaction, it may cost too much
time to write binlog cache and block all subsequent transactions for
a long time. One possible solution of this issue is to write the binlog
of the large transaction to a temporary file and then rename it to the
next new binlog file.
We need to reserve some space for format description event, gtid list
event, binlog checkpoint event and gtid event. After writing the binlog
cache, we lock LOCK_log and write the aforementioned events to the
reserved space. The default size of reserved space is 10MB, any unusable
space is filled with a skip event(Ignorable_log_event). If any error
occurs before acquiring LOCK_log, we fallback to the original commit
path. However, if an error occurs after acquiring LOCK_log, we abort
the server.
New variable: non_blocking_binlog_threshold, if the size of binlog cache
exceeds this value, try using non-blocking binlog.
Limitations:
Binlog encryption, wsrep transaction, stmt cache, incident event and
slave thread are not supported.
---
mysql-test/main/mysqld--help.result | 5 +
.../binlog/r/non_blocking_binlog_basic.result | 42 +++
.../binlog/r/non_blocking_binlog_error.result | 36 ++
.../binlog/t/non_blocking_binlog_basic.test | 31 ++
.../binlog/t/non_blocking_binlog_error.test | 49 +++
.../perfschema/r/dml_setup_instruments.result | 2 +-
.../rpl/r/rpl_non_blocking_binlog.result | 22 ++
.../suite/rpl/t/rpl_non_blocking_binlog.test | 31 ++
.../r/sysvars_server_notembedded.result | 10 +
sql/log.cc | 354 +++++++++++++++++-
sql/log.h | 30 +-
sql/mysqld.cc | 9 +-
sql/mysqld.h | 4 +-
sql/sys_vars.cc | 8 +
14 files changed, 615 insertions(+), 18 deletions(-)
create mode 100644 mysql-test/suite/binlog/r/non_blocking_binlog_basic.result
create mode 100644 mysql-test/suite/binlog/r/non_blocking_binlog_error.result
create mode 100644 mysql-test/suite/binlog/t/non_blocking_binlog_basic.test
create mode 100644 mysql-test/suite/binlog/t/non_blocking_binlog_error.test
create mode 100644 mysql-test/suite/rpl/r/rpl_non_blocking_binlog.result
create mode 100644 mysql-test/suite/rpl/t/rpl_non_blocking_binlog.test
diff --git a/mysql-test/main/mysqld--help.result b/mysql-test/main/mysqld--help.result
index 431c37c25ed..d652eacadb0 100644
--- a/mysql-test/main/mysqld--help.result
+++ b/mysql-test/main/mysqld--help.result
@@ -735,6 +735,10 @@ The following specify which files/extra groups are read (specified before remain
--net-write-timeout=#
Number of seconds to wait for a block to be written to a
connection before aborting the write
+ --non-blocking-binlog-threshold[=#]
+ If the binlog size of a transaction exceeds this value,
+ we write it to a new temporary file and rename it to the
+ next binlog file.
--note-verbosity=name
Verbosity level for note-warnings given to the user. See
also @@sql_notes.. Any combination of: basic,
@@ -1797,6 +1801,7 @@ net-buffer-length 16384
net-read-timeout 30
net-retry-count 10
net-write-timeout 60
+non-blocking-binlog-threshold 18446744073709551615
note-verbosity basic,explain
old FALSE
old-mode UTF8_IS_UTF8MB3
diff --git a/mysql-test/suite/binlog/r/non_blocking_binlog_basic.result b/mysql-test/suite/binlog/r/non_blocking_binlog_basic.result
new file mode 100644
index 00000000000..dfbf6721f89
--- /dev/null
+++ b/mysql-test/suite/binlog/r/non_blocking_binlog_basic.result
@@ -0,0 +1,42 @@
+set global non_blocking_binlog_threshold=1;
+Warnings:
+Warning 1292 Truncated incorrect non_blocking_binlog_threshold value: '1'
+show variables like 'non_blocking_binlog_threshold';
+Variable_name Value
+non_blocking_binlog_threshold 268435456
+set global non_blocking_binlog_threshold=256*1024*1024;
+show variables like 'non_blocking_binlog_threshold';
+Variable_name Value
+non_blocking_binlog_threshold 268435456
+set global non_blocking_binlog_threshold=default;
+show variables like 'non_blocking_binlog_threshold';
+Variable_name Value
+non_blocking_binlog_threshold 18446744073709551615
+reset master;
+create table t1(id int, c1 int) engine=innodb;
+insert into t1 values(1,1);
+set debug_dbug='+d,non_blocking_binlog_ignore_cache_size';
+update t1 set c1=2;
+set debug_dbug='-d,non_blocking_binlog_ignore_cache_size';
+insert into t1 values(3,3);
+show master status;
+File Position Binlog_Do_DB Binlog_Ignore_DB
+master-bin.000002 #
+show binlog events in 'master-bin.000002';
+Log_name Pos Event_type Server_id End_log_pos Info
+master-bin.000002 # Format_desc # # #
+master-bin.000002 # Gtid_list # # #
+master-bin.000002 # Binlog_checkpoint # # #
+master-bin.000002 # Gtid # # #
+master-bin.000002 # Ignorable log event # # #
+master-bin.000002 # Annotate_rows # # #
+master-bin.000002 # Table_map # # #
+master-bin.000002 # Update_rows_v1 # # #
+master-bin.000002 # Xid # # #
+master-bin.000002 # Gtid # # #
+master-bin.000002 # Annotate_rows # # #
+master-bin.000002 # Table_map # # #
+master-bin.000002 # Write_rows_v1 # # #
+master-bin.000002 # Xid # # #
+drop table t1;
+include/assert_grep.inc [write non-blocking binlog succeed should be logged]
diff --git a/mysql-test/suite/binlog/r/non_blocking_binlog_error.result b/mysql-test/suite/binlog/r/non_blocking_binlog_error.result
new file mode 100644
index 00000000000..536c80e04b4
--- /dev/null
+++ b/mysql-test/suite/binlog/r/non_blocking_binlog_error.result
@@ -0,0 +1,36 @@
+call mtr.add_suppression(".*");
+reset master;
+create table t1(id int, c1 int) engine=innodb;
+insert into t1 values(1,1);
+set debug_dbug='+d,non_blocking_binlog_ignore_cache_size';
+set debug_dbug='+d,non_blocking_binlog_open_file_failed';
+update t1 set c1=2;
+set debug_dbug='-d,non_blocking_binlog_ignore_cache_size';
+set debug_dbug='-d,non_blocking_binlog_open_file_failed';
+insert into t1 values(3,3);
+show master status;
+File Position Binlog_Do_DB Binlog_Ignore_DB
+master-bin.000001 #
+include/assert_grep.inc [Open file failed should be logged]
+set debug_dbug='+d,non_blocking_binlog_ignore_cache_size';
+set debug_dbug='+d,non_blocking_binlog_write_cache_failed';
+update t1 set c1=3;
+set debug_dbug='-d,non_blocking_binlog_ignore_cache_size';
+set debug_dbug='-d,non_blocking_binlog_write_cache_failed';
+insert into t1 values(4,4);
+show master status;
+File Position Binlog_Do_DB Binlog_Ignore_DB
+master-bin.000001 #
+set debug_dbug='+d,non_blocking_binlog_ignore_cache_size';
+set debug_dbug='+d,non_blocking_binlog_write_gtid_failed';
+update t1 set c1=5;
+ERROR HY000: Lost connection to server during query
+show master status;
+File Position Binlog_Do_DB Binlog_Ignore_DB
+master-bin.000002 #
+select * from t1;
+id c1
+1 3
+3 3
+4 4
+drop table t1;
diff --git a/mysql-test/suite/binlog/t/non_blocking_binlog_basic.test b/mysql-test/suite/binlog/t/non_blocking_binlog_basic.test
new file mode 100644
index 00000000000..2964f921698
--- /dev/null
+++ b/mysql-test/suite/binlog/t/non_blocking_binlog_basic.test
@@ -0,0 +1,31 @@
+--source include/have_debug.inc
+--source include/have_innodb.inc
+--source include/have_binlog_format_row.inc
+--source include/not_embedded.inc
+
+set global non_blocking_binlog_threshold=1;
+show variables like 'non_blocking_binlog_threshold';
+set global non_blocking_binlog_threshold=256*1024*1024;
+show variables like 'non_blocking_binlog_threshold';
+set global non_blocking_binlog_threshold=default;
+show variables like 'non_blocking_binlog_threshold';
+
+reset master;
+create table t1(id int, c1 int) engine=innodb;
+insert into t1 values(1,1);
+set debug_dbug='+d,non_blocking_binlog_ignore_cache_size';
+update t1 set c1=2;
+set debug_dbug='-d,non_blocking_binlog_ignore_cache_size';
+insert into t1 values(3,3);
+--replace_column 2 #
+show master status;
+--replace_column 2 # 4 # 5 # 6 #
+show binlog events in 'master-bin.000002';
+drop table t1;
+
+--let $assert_count = 1
+--let $assert_file = $MYSQLTEST_VARDIR/log/mysqld.1.err
+--let $assert_text = write non-blocking binlog succeed should be logged
+--let $assert_select = write non-blocking binlog succeed
+--let $assert_only_after = CURRENT_TEST
+--source include/assert_grep.inc
diff --git a/mysql-test/suite/binlog/t/non_blocking_binlog_error.test b/mysql-test/suite/binlog/t/non_blocking_binlog_error.test
new file mode 100644
index 00000000000..6699006bffb
--- /dev/null
+++ b/mysql-test/suite/binlog/t/non_blocking_binlog_error.test
@@ -0,0 +1,49 @@
+--source include/have_debug.inc
+--source include/have_innodb.inc
+--source include/have_binlog_format_row.inc
+--source include/not_valgrind.inc
+--source include/not_embedded.inc
+
+call mtr.add_suppression(".*");
+
+reset master;
+create table t1(id int, c1 int) engine=innodb;
+insert into t1 values(1,1);
+set debug_dbug='+d,non_blocking_binlog_ignore_cache_size';
+set debug_dbug='+d,non_blocking_binlog_open_file_failed';
+update t1 set c1=2;
+set debug_dbug='-d,non_blocking_binlog_ignore_cache_size';
+set debug_dbug='-d,non_blocking_binlog_open_file_failed';
+insert into t1 values(3,3);
+--replace_column 2 #
+show master status;
+
+--let $assert_count = 1
+--let $assert_file = $MYSQLTEST_VARDIR/log/mysqld.1.err
+--let $assert_text = Open file failed should be logged
+--let $assert_select = Open file .* failed
+--let $assert_only_after = CURRENT_TEST
+--source include/assert_grep.inc
+
+set debug_dbug='+d,non_blocking_binlog_ignore_cache_size';
+set debug_dbug='+d,non_blocking_binlog_write_cache_failed';
+update t1 set c1=3;
+set debug_dbug='-d,non_blocking_binlog_ignore_cache_size';
+set debug_dbug='-d,non_blocking_binlog_write_cache_failed';
+insert into t1 values(4,4);
+--replace_column 2 #
+show master status;
+
+set debug_dbug='+d,non_blocking_binlog_ignore_cache_size';
+set debug_dbug='+d,non_blocking_binlog_write_gtid_failed';
+--exec echo "restart" > $MYSQLTEST_VARDIR/tmp/mysqld.1.expect
+--error 2013
+update t1 set c1=5;
+--enable_reconnect
+--source include/wait_until_connected_again.inc
+--replace_column 2 #
+show master status;
+# c1 should not be 5
+select * from t1;
+
+drop table t1;
diff --git a/mysql-test/suite/perfschema/r/dml_setup_instruments.result b/mysql-test/suite/perfschema/r/dml_setup_instruments.result
index ff000a09312..95d8bfb882a 100644
--- a/mysql-test/suite/perfschema/r/dml_setup_instruments.result
+++ b/mysql-test/suite/perfschema/r/dml_setup_instruments.result
@@ -21,6 +21,7 @@ where name like 'Wait/Synch/Rwlock/sql/%'
'wait/synch/rwlock/sql/LOCK_named_pipe_full_access_group')
order by name limit 10;
NAME ENABLED TIMED
+wait/synch/rwlock/sql/binlog_checksum_rwlock YES YES
wait/synch/rwlock/sql/LOCK_all_status_vars YES YES
wait/synch/rwlock/sql/LOCK_dbnames YES YES
wait/synch/rwlock/sql/LOCK_dboptions YES YES
@@ -30,7 +31,6 @@ wait/synch/rwlock/sql/LOCK_SEQUENCE YES YES
wait/synch/rwlock/sql/LOCK_ssl_refresh YES YES
wait/synch/rwlock/sql/LOCK_system_variables_hash YES YES
wait/synch/rwlock/sql/LOCK_sys_init_connect YES YES
-wait/synch/rwlock/sql/LOCK_sys_init_slave YES YES
select * from performance_schema.setup_instruments
where name like 'Wait/Synch/Cond/sql/%'
and name not in (
diff --git a/mysql-test/suite/rpl/r/rpl_non_blocking_binlog.result b/mysql-test/suite/rpl/r/rpl_non_blocking_binlog.result
new file mode 100644
index 00000000000..c130848bfe9
--- /dev/null
+++ b/mysql-test/suite/rpl/r/rpl_non_blocking_binlog.result
@@ -0,0 +1,22 @@
+include/master-slave.inc
+[connection master]
+connection master;
+create table t1(id int, c1 int) engine=innodb;
+insert into t1 values(1,1);
+set debug_dbug='+d,non_blocking_binlog_ignore_cache_size';
+update t1 set c1=2;
+set debug_dbug='-d,non_blocking_binlog_ignore_cache_size';
+include/assert_grep.inc [write non-blocking binlog succeed should be logged]
+insert into t1 values(3,3);
+checksum table t1;
+Table Checksum
+test.t1 4217795292
+include/sync_slave_sql_with_master.inc
+connection slave;
+checksum table t1;
+Table Checksum
+test.t1 4217795292
+connection master;
+drop table t1;
+include/sync_slave_sql_with_master.inc
+include/rpl_end.inc
diff --git a/mysql-test/suite/rpl/t/rpl_non_blocking_binlog.test b/mysql-test/suite/rpl/t/rpl_non_blocking_binlog.test
new file mode 100644
index 00000000000..558ca63c5ec
--- /dev/null
+++ b/mysql-test/suite/rpl/t/rpl_non_blocking_binlog.test
@@ -0,0 +1,31 @@
+--source include/have_debug.inc
+--source include/have_innodb.inc
+--source include/master-slave.inc
+--source include/not_embedded.inc
+
+connection master;
+create table t1(id int, c1 int) engine=innodb;
+insert into t1 values(1,1);
+set debug_dbug='+d,non_blocking_binlog_ignore_cache_size';
+update t1 set c1=2;
+set debug_dbug='-d,non_blocking_binlog_ignore_cache_size';
+
+--let $assert_count = 1
+--let $assert_file = $MYSQLTEST_VARDIR/log/mysqld.1.err
+--let $assert_text = write non-blocking binlog succeed should be logged
+--let $assert_select = write non-blocking binlog succeed
+--let $assert_only_after = CURRENT_TEST
+--source include/assert_grep.inc
+
+insert into t1 values(3,3);
+checksum table t1;
+
+--source include/sync_slave_sql_with_master.inc
+connection slave;
+checksum table t1;
+
+connection master;
+drop table t1;
+--source include/sync_slave_sql_with_master.inc
+
+--source include/rpl_end.inc
diff --git a/mysql-test/suite/sys_vars/r/sysvars_server_notembedded.result b/mysql-test/suite/sys_vars/r/sysvars_server_notembedded.result
index 2740319dd05..0ad02a33089 100644
--- a/mysql-test/suite/sys_vars/r/sysvars_server_notembedded.result
+++ b/mysql-test/suite/sys_vars/r/sysvars_server_notembedded.result
@@ -2512,6 +2512,16 @@ NUMERIC_BLOCK_SIZE 1
ENUM_VALUE_LIST NULL
READ_ONLY NO
COMMAND_LINE_ARGUMENT REQUIRED
+VARIABLE_NAME NON_BLOCKING_BINLOG_THRESHOLD
+VARIABLE_SCOPE GLOBAL
+VARIABLE_TYPE BIGINT UNSIGNED
+VARIABLE_COMMENT If the binlog size of a transaction exceeds this value, we write it to a new temporary file and rename it to the next binlog file.
+NUMERIC_MIN_VALUE 268435456
+NUMERIC_MAX_VALUE 18446744073709551615
+NUMERIC_BLOCK_SIZE 1
+ENUM_VALUE_LIST NULL
+READ_ONLY NO
+COMMAND_LINE_ARGUMENT OPTIONAL
VARIABLE_NAME NOTE_VERBOSITY
VARIABLE_SCOPE SESSION
VARIABLE_TYPE SET
diff --git a/sql/log.cc b/sql/log.cc
index 460cefea47b..9de1c6118cd 100644
--- a/sql/log.cc
+++ b/sql/log.cc
@@ -74,6 +74,9 @@
#include <utility> // pair
#endif
+#include <atomic>
+#include <chrono>
+
/* max size of the log message */
#define MAX_LOG_BUFFER_SIZE 1024
#define MAX_TIME_SIZE 32
@@ -3727,7 +3730,10 @@ bool MYSQL_BIN_LOG::open(const char *log_name,
enum cache_type io_cache_type_arg,
ulong max_size_arg,
bool null_created_arg,
- bool need_mutex)
+ bool need_mutex,
+ const char *file_to_rename,
+ my_off_t file_size,
+ group_commit_entry *entry)
{
xid_count_per_binlog *new_xid_list_entry= NULL, *b;
DBUG_ENTER("MYSQL_BIN_LOG::open");
@@ -3792,6 +3798,18 @@ bool MYSQL_BIN_LOG::open(const char *log_name,
write_error= 0;
+ if (file_to_rename)
+ {
+ DBUG_ASSERT(!is_relay_log && entry);
+ if (mysql_file_rename(PSI_NOT_INSTRUMENTED, file_to_rename, log_file_name,
+ MY_WME | MY_NABP | MY_WAIT_IF_FULL) != 0)
+ {
+ sql_print_error("rename from '%s' to '%s' failed!", file_to_rename,
+ log_file_name);
+ DBUG_RETURN(1);
+ }
+ }
+
/* open the main log file */
if (MYSQL_LOG::open(
#ifdef HAVE_PSI_INTERFACE
@@ -3816,7 +3834,7 @@ bool MYSQL_BIN_LOG::open(const char *log_name,
{
bool write_file_name_to_index_file=0;
- if (!my_b_filelength(&log_file))
+ if (!my_b_filelength(&log_file) || file_to_rename)
{
/*
The binary log file was empty (probably newly created)
@@ -3993,6 +4011,12 @@ bool MYSQL_BIN_LOG::open(const char *log_name,
goto err;
bytes_written+= description_event_for_queue->data_written;
}
+ if (file_to_rename)
+ {
+ if (write_gtid_and_skip_event(file_size, entry))
+ goto err;
+ }
+
if (flush_io_cache(&log_file) ||
mysql_file_sync(log_file.file, MYF(MY_WME)))
goto err;
@@ -6826,7 +6850,6 @@ MYSQL_BIN_LOG::write_gtid_event(THD *thd, bool standalone,
thd->variables.server_id= global_system_variables.server_id;
}
#endif
-
if (write_event(>id_event))
DBUG_RETURN(true);
status_var_add(thd->status_var.binlog_bytes_written, gtid_event.data_written);
@@ -7849,11 +7872,18 @@ int Event_log::write_cache_raw(THD *thd, IO_CACHE *cache)
events prior to fill in the binlog cache.
*/
-int Event_log::write_cache(THD *thd, binlog_cache_data *cache_data)
+int Event_log::write_cache(THD *thd, binlog_cache_data *cache_data,
+ IO_CACHE *f)
{
DBUG_ENTER("Event_log::write_cache");
IO_CACHE *cache= &cache_data->cache_log;
- mysql_mutex_assert_owner(&LOCK_log);
+
+ IO_CACHE *out_file= f;
+ if (likely(f == nullptr))
+ {
+ mysql_mutex_assert_owner(&LOCK_log);
+ out_file= get_log_file();
+ }
/*
If possible, just copy the cache over byte-by-byte with pre-computed
@@ -7863,7 +7893,7 @@ int Event_log::write_cache(THD *thd, binlog_cache_data *cache_data)
likely(!crypto.scheme) &&
likely(!opt_binlog_legacy_event_pos))
{
- int res= my_b_copy_all_to_cache(cache, &log_file);
+ int res= my_b_copy_all_to_cache(cache, out_file);
status_var_add(thd->status_var.binlog_bytes_written, my_b_tell(cache));
DBUG_RETURN(res ? ER_ERROR_ON_WRITE : 0);
}
@@ -7873,7 +7903,7 @@ int Event_log::write_cache(THD *thd, binlog_cache_data *cache_data)
/* Amount of remaining bytes in the IO_CACHE read buffer. */
size_t log_file_pos;
uchar header_buf[LOG_EVENT_HEADER_LEN];
- Log_event_writer writer(get_log_file(), 0,
+ Log_event_writer writer(out_file, 0,
(enum_binlog_checksum_alg)binlog_checksum_options,
&crypto);
uint checksum_len= writer.checksum_len;
@@ -7904,7 +7934,7 @@ int Event_log::write_cache(THD *thd, binlog_cache_data *cache_data)
split.
*/
- log_file_pos= (size_t)my_b_tell(get_log_file());
+ log_file_pos= (size_t)my_b_tell(out_file);
for (;;)
{
/*
@@ -8194,6 +8224,7 @@ MYSQL_BIN_LOG::write_transaction_to_binlog(THD *thd,
ha_info= all ? thd->transaction->all.ha_list : thd->transaction->stmt.ha_list;
entry.ro_1pc= is_ro_1pc;
entry.end_event= end_ev;
+ entry.non_blocking_log= false;
auto has_xid= entry.end_event->get_type_code() == XID_EVENT;
for (; has_xid && !entry.need_unlog && ha_info; ha_info= ha_info->next())
@@ -8581,7 +8612,23 @@ MYSQL_BIN_LOG::queue_for_group_commit(group_commit_entry *orig_entry)
bool
MYSQL_BIN_LOG::write_transaction_to_binlog_events(group_commit_entry *entry)
{
- int is_leader= queue_for_group_commit(entry);
+ int is_leader;
+ if (unlikely(entry->cache_mngr->trx_cache.get_byte_position() >=
+ non_blocking_binlog_threshold) ||
+ DBUG_IF("non_blocking_binlog_ignore_cache_size"))
+ {
+ if (!can_use_non_blocking_binlog(entry) ||
+ write_non_blocking_binlog(entry))
+ goto original_commit_path;
+ /* thread using non-blocking binlog is treated as a single group */
+ is_leader= 1;
+ entry->non_blocking_log= true;
+ entry->next= nullptr;
+ goto group_commit_leader;
+ }
+original_commit_path:
+
+ is_leader= queue_for_group_commit(entry);
#ifdef WITH_WSREP
/* commit order was released in queue_for_group_commit() call,
here we check if wsrep_commit_ordered() failed or if we are leader */
@@ -8616,7 +8663,10 @@ MYSQL_BIN_LOG::write_transaction_to_binlog_events(group_commit_entry *entry)
if (is_leader < 0)
return true; /* Error */
else if (is_leader)
+ {
+group_commit_leader:
trx_group_commit_leader(entry);
+ }
else if (!entry->queued_by_other)
{
DEBUG_SYNC(entry->thd, "after_semisync_queue");
@@ -8748,6 +8798,12 @@ MYSQL_BIN_LOG::trx_group_commit_leader(group_commit_entry *leader)
("commit_before_get_LOCK_log SIGNAL waiting WAIT_FOR cont TIMEOUT 1")));
);
#endif
+ if (unlikely(leader->non_blocking_log))
+ {
+ mysql_mutex_assert_owner(&LOCK_log);
+ current= leader;
+ goto after_fetching_group;
+ }
/*
Lock the LOCK_log(), and once we get it, collect any additional writes
that queued up while we were waiting.
@@ -8766,6 +8822,7 @@ MYSQL_BIN_LOG::trx_group_commit_leader(group_commit_entry *leader)
current= group_commit_queue;
group_commit_queue= NULL;
mysql_mutex_unlock(&LOCK_prepare_ordered);
+after_fetching_group:
binlog_id= current_binlog_id;
/* As the queue is in reverse order of entering, reverse it. */
@@ -8824,9 +8881,12 @@ MYSQL_BIN_LOG::trx_group_commit_leader(group_commit_entry *leader)
!cache_mngr->trx_cache.empty() ||
current->thd->transaction->xid_state.is_explicit_XA());
- if (unlikely((current->error= write_transaction_or_stmt(current,
- commit_id))))
- current->commit_errno= errno;
+ if (likely(leader->non_blocking_log == false))
+ {
+ if (unlikely((current->error=
+ write_transaction_or_stmt(current, commit_id))))
+ current->commit_errno= errno;
+ }
strmake_buf(cache_mngr->last_commit_pos_file, log_file_name);
commit_offset= my_b_write_tell(&log_file);
@@ -12459,6 +12519,7 @@ mysql_bin_log_commit_pos(THD *thd, ulonglong *out_pos, const char **out_file)
}
#endif /* INNODB_COMPATIBILITY_HOOKS */
+mysql_rwlock_t binlog_checksum_rwlock;
static void
binlog_checksum_update(MYSQL_THD thd, struct st_mysql_sys_var *var,
@@ -12468,6 +12529,7 @@ binlog_checksum_update(MYSQL_THD thd, struct st_mysql_sys_var *var,
bool check_purge= false;
ulong UNINIT_VAR(prev_binlog_id);
+ mysql_rwlock_wrlock(&binlog_checksum_rwlock);
mysql_mutex_lock(mysql_bin_log.get_log_lock());
if(mysql_bin_log.is_open())
{
@@ -12484,6 +12546,7 @@ binlog_checksum_update(MYSQL_THD thd, struct st_mysql_sys_var *var,
DBUG_ASSERT(binlog_checksum_options == value);
mysql_bin_log.checksum_alg_reset= BINLOG_CHECKSUM_ALG_UNDEF;
mysql_mutex_unlock(mysql_bin_log.get_log_lock());
+ mysql_rwlock_unlock(&binlog_checksum_rwlock);
if (check_purge)
mysql_bin_log.checkpoint_and_purge(prev_binlog_id);
}
@@ -12781,3 +12844,270 @@ void wsrep_register_binlog_handler(THD *thd, bool trx)
}
#endif /* WITH_WSREP */
+
+bool MYSQL_BIN_LOG::can_use_non_blocking_binlog(group_commit_entry *entry)
+{
+ DBUG_ASSERT(entry->cache_mngr->trx_cache.get_byte_position() >=
+ non_blocking_binlog_threshold ||
+ DBUG_IF("non_blocking_binlog_ignore_cache_size"));
+ THD *thd= entry->thd;
+ if (unlikely(!is_open()) || encrypt_binlog ||
+ !entry->cache_mngr->stmt_cache.empty() ||
+ entry->cache_mngr->trx_cache.has_incident() || thd->slave_thread ||
+ thd->wait_for_commit_ptr)
+ return false;
+#ifdef WITH_WSREP
+ if (wsrep_is_active(thd))
+ return false;
+#endif /* WITH_WSREP */
+ return true;
+}
+
+std::string MYSQL_BIN_LOG::generate_random_file_name()
+{
+ static std::atomic<uint64_t> temp_bin_counter{0};
+ static std::atomic<bool> binlog_dir_inited{false};
+ static std::string binlog_dir;
+ std::string temp_file_name;
+ if (unlikely(!binlog_dir_inited.load(std::memory_order_acquire)))
+ {
+ char dev[FN_REFLEN];
+ size_t dev_length;
+ mysql_mutex_lock(&LOCK_log);
+ if (!binlog_dir_inited.load(std::memory_order_relaxed) && name != nullptr)
+ {
+ /*
+ see MYSQL_BIN_LOG::generate_new_name and fn_format for more
+ information
+ */
+ size_t length= dirname_part(dev, name, &dev_length);
+ if (length == 0)
+ {
+ // log_bin is not set, use mysql_data_home
+ convert_dirname(dev, mysql_data_home, NullS);
+ }
+ unpack_dirname(dev, dev); // Replace ~/.. with dir
+ binlog_dir.assign(dev);
+ binlog_dir_inited.store(true);
+ }
+ mysql_mutex_unlock(&LOCK_log);
+ }
+
+ temp_file_name.reserve(FN_REFLEN);
+ auto now_in_sys= std::chrono::system_clock::now().time_since_epoch();
+ auto now_in_ms=
+ std::chrono::duration_cast<std::chrono::milliseconds>(now_in_sys)
+ .count();
+ auto count= temp_bin_counter.fetch_add(1);
+ temp_file_name.append(binlog_dir);
+ temp_file_name.append("_temp_bin_");
+ temp_file_name.append(std::to_string(now_in_ms));
+ temp_file_name.push_back('_');
+ temp_file_name.append(std::to_string(count));
+
+ return temp_file_name;
+}
+
+void MYSQL_BIN_LOG::generate_skip_event(uchar *buf, size_t buf_len,
+ my_off_t pos, THD *thd)
+{
+ uchar *header= buf;
+ my_time_t now= thd->start_time;
+ int4store(header, now);
+ header[EVENT_TYPE_OFFSET]= IGNORABLE_LOG_EVENT;
+ int4store(header + SERVER_ID_OFFSET, server_id);
+ int4store(header + EVENT_LEN_OFFSET, buf_len);
+ int4store(header + LOG_POS_OFFSET, pos + buf_len);
+ int2store(header + FLAGS_OFFSET, LOG_EVENT_IGNORABLE_F);
+ if (binlog_checksum_options != BINLOG_CHECKSUM_ALG_OFF &&
+ binlog_checksum_options != BINLOG_CHECKSUM_ALG_UNDEF)
+ {
+ ha_checksum crc= my_checksum(0, header, buf_len - BINLOG_CHECKSUM_LEN);
+ int4store(header + buf_len - BINLOG_CHECKSUM_LEN, crc);
+ }
+}
+
+template <typename F= std::function<void()>> class Scoped_guard
+{
+public:
+ Scoped_guard(F f);
+ ~Scoped_guard();
+
+private:
+ F func;
+};
+
+template <typename F> Scoped_guard<F>::Scoped_guard(F f) : func(f) {}
+
+template <typename F> Scoped_guard<F>::~Scoped_guard()
+{
+ if (func)
+ func();
+}
+
+bool MYSQL_BIN_LOG::write_gtid_and_skip_event(my_off_t file_size,
+ group_commit_entry *entry)
+{
+ mysql_mutex_assert_owner(&LOCK_log);
+ DBUG_EXECUTE_IF("non_blocking_binlog_write_gtid_failed", { return true; });
+ uchar *skip_event_buf= nullptr;
+ Scoped_guard<> scoped_guard{[&]() {
+ if (skip_event_buf)
+ my_free(skip_event_buf);
+ }};
+ bool has_xid= entry->end_event->get_type_code() == XID_EVENT;
+ if (write_gtid_event(entry->thd, is_prepared_xa(entry->thd),
+ entry->using_trx_cache, 0, has_xid, entry->ro_1pc))
+ return true;
+ my_off_t skip_event_start= my_b_safe_tell(&log_file);
+ size_t skip_event_len= non_blocking_binlog_reserved_size - skip_event_start;
+ skip_event_buf=
+ (uchar *) my_malloc(PSI_INSTRUMENT_ME, skip_event_len, MYF(MY_ZEROFILL));
+ generate_skip_event(skip_event_buf, skip_event_len, skip_event_start,
+ entry->thd);
+ if (my_b_write(&log_file, skip_event_buf, skip_event_len) != 0)
+ return true;
+ DBUG_ASSERT(non_blocking_binlog_reserved_size == my_b_safe_tell(&log_file));
+ if (flush_io_cache(&log_file) != 0)
+ return true;
+ if (reinit_io_cache(&log_file, io_cache_type, file_size, FALSE, TRUE))
+ return true;
+ return false;
+}
+
+bool MYSQL_BIN_LOG::write_non_blocking_binlog(group_commit_entry *entry)
+{
+ /* generate random file name */
+ std::string temp_file_name(generate_random_file_name());
+ if (temp_file_name.empty())
+ {
+ sql_print_warning("Generate random file name failed!");
+ return true;
+ }
+ if (temp_file_name.size() >= FN_REFLEN)
+ {
+ sql_print_warning("The name of temp file '%s' is too long!",
+ temp_file_name.c_str());
+ return true;
+ }
+ THD *thd= entry->thd;
+ IO_CACHE temp_log_file;
+ File file= -1;
+ bool io_cache_inited= false;
+ bool delete_file= true;
+ bool binlog_checksum_locked= false;
+ // ignore all errors in this function
+ Dummy_error_handler dummy_error_handler;
+ thd->push_internal_handler(&dummy_error_handler);
+ Scoped_guard<> scoped_guard{[&]() {
+ if (io_cache_inited)
+ end_io_cache(&temp_log_file);
+ if (file > 0)
+ {
+ mysql_file_close(file, MYF(MY_WME));
+ if (delete_file)
+ {
+ mysql_file_delete(PSI_NOT_INSTRUMENTED, temp_file_name.c_str(),
+ MYF(MY_WME));
+ }
+ }
+ if (binlog_checksum_locked)
+ mysql_rwlock_unlock(&binlog_checksum_rwlock);
+ thd->pop_internal_handler();
+ }};
+
+ // open temporary file
+ int flags= O_WRONLY | O_CREAT;
+ DBUG_EXECUTE_IF("non_blocking_binlog_open_file_failed",
+ { flags= O_WRONLY; });
+ if ((file= mysql_file_open(PSI_NOT_INSTRUMENTED, temp_file_name.c_str(),
+ flags, MYF(MY_WME))) < 0)
+ {
+ sql_print_error("Open file '%s' failed!", temp_file_name.c_str());
+ return true;
+ }
+ if (init_io_cache(&temp_log_file, file, LOG_BIN_IO_SIZE, WRITE_CACHE,
+ non_blocking_binlog_reserved_size, 0,
+ MYF(MY_WME | MY_NABP | MY_WAIT_IF_FULL)))
+ {
+ sql_print_error("Init io cache for file '%s' failed!",
+ temp_file_name.c_str());
+ return true;
+ }
+ io_cache_inited= true;
+ // prevent any changes to binlog_checksum_options while writing cache
+ mysql_rwlock_rdlock(&binlog_checksum_rwlock);
+ binlog_checksum_locked= true;
+ enum_binlog_checksum_alg alg=
+ (enum_binlog_checksum_alg) binlog_checksum_options;
+ // write cache
+ {
+ binlog_cache_mngr *mngr= entry->cache_mngr;
+ binlog_cache_data *cache_data= mngr->get_binlog_cache_data(TRUE);
+ IO_CACHE *cache= &cache_data->cache_log;
+ if (reinit_io_cache(cache, READ_CACHE, 0, 0, 0))
+ return true;
+ if (write_cache(entry->thd, cache_data, &temp_log_file) ||
+ DBUG_IF("non_blocking_binlog_write_cache_failed"))
+ return true;
+ if (write_event(entry->end_event, alg, nullptr, &temp_log_file))
+ return true;
+ status_var_add(entry->thd->status_var.binlog_bytes_written,
+ entry->end_event->data_written);
+ if (unlikely(cache->error))
+ return true;
+ }
+ if (flush_io_cache(&temp_log_file) != 0)
+ return true;
+ mysql_mutex_lock(&LOCK_log);
+ mysql_mutex_lock(&LOCK_index);
+ // rename temporary file to next binlog file, abort if any error occurs
+ {
+ char new_name[FN_REFLEN], *new_name_ptr, *old_name;
+ File UNINIT_VAR(old_file);
+ // Reuse old name if not binlog and not update log
+ new_name_ptr= name;
+ if (unlikely(generate_new_name(new_name, name, 0)))
+ abort();
+ new_name_ptr= new_name;
+ /*
+ We log the whole file name for log file as the user may decide
+ to change base names at some point.
+ */
+ Rotate_log_event r(new_name + dirname_length(new_name), 0,
+ LOG_EVENT_OFFSET, 0);
+ if (unlikely(write_event(&r, alg)))
+ abort();
+ if (unlikely(flush_io_cache(&log_file) != 0))
+ abort();
+ update_binlog_end_pos();
+ old_name= name;
+ name= 0; // Don't free name
+ old_file= log_file.file;
+ uint close_flag=
+ LOG_CLOSE_TO_BE_OPENED | LOG_CLOSE_INDEX | LOG_CLOSE_DELAYED_CLOSE;
+ /*
+ We need to keep the old binlog file open (and marked as in-use) until
+ the new one is fully created and synced to disk and index. Otherwise we
+ leave a window where if we crash, there is no binlog file marked as
+ crashed for server restart to detect the need for recovery.
+ */
+ close(close_flag);
+ if (unlikely(open_index_file(index_file_name, 0, FALSE)))
+ abort();
+ if (unlikely(open(old_name, new_name_ptr, 0, io_cache_type, max_size, 1,
+ FALSE, temp_file_name.c_str(),
+ my_b_safe_tell(&temp_log_file), entry)))
+ abort();
+ my_free(old_name);
+ delete_file= false;
+ clear_inuse_flag_when_closing(old_file);
+ mysql_file_close(old_file, MYF(MY_WME));
+ }
+ mysql_mutex_unlock(&LOCK_index);
+ // unlock lock_log in trx_group_commit_leader
+#ifndef NDEBUG
+ sql_print_information("write non-blocking binlog succeed!");
+#endif /* ! NDEBUG */
+ return false;
+}
diff --git a/sql/log.h b/sql/log.h
index fc5209d1922..af2b56da802 100644
--- a/sql/log.h
+++ b/sql/log.h
@@ -19,6 +19,7 @@
#include "handler.h" /* my_xid */
#include "rpl_constants.h"
+#include <string>
class Relay_log_info;
class Gtid_index_writer;
@@ -406,7 +407,8 @@ class Event_log: public MYSQL_LOG
void set_write_error(THD *thd, bool is_transactional);
static bool check_write_error(THD *thd);
static bool check_cache_error(THD *thd, binlog_cache_data *cache_data);
- int write_cache(THD *thd, binlog_cache_data *cache_data);
+ int write_cache(THD *thd, binlog_cache_data *cache_data,
+ IO_CACHE *f= nullptr);
int write_cache_raw(THD *thd, IO_CACHE *cache);
char* get_name() { return name; }
void cleanup()
@@ -650,6 +652,7 @@ class MYSQL_BIN_LOG: public TC_LOG, private Event_log
bool queued_by_other;
ulong binlog_id;
bool ro_1pc; // passes the binlog_cache_mngr::ro_1pc value to Gtid ctor
+ bool non_blocking_log;
};
/*
@@ -986,7 +989,10 @@ class MYSQL_BIN_LOG: public TC_LOG, private Event_log
enum cache_type io_cache_type_arg,
ulong max_size,
bool null_created,
- bool need_mutex);
+ bool need_mutex,
+ const char *file_to_rename= nullptr,
+ my_off_t file_size= 0,
+ group_commit_entry *entry= nullptr);
bool open_index_file(const char *index_file_name_arg,
const char *log_name, bool need_mutex);
/* Use this to start writing a new log file */
@@ -1177,6 +1183,24 @@ class MYSQL_BIN_LOG: public TC_LOG, private Event_log
*/
my_off_t binlog_end_pos;
char binlog_end_pos_file[FN_REFLEN];
+
+private:
+ // reserved size for format_description_log_event, gtid_list_log_event ...
+ static const my_off_t non_blocking_binlog_reserved_size= 10 * 1024 * 1024;
+ bool can_use_non_blocking_binlog(group_commit_entry *entry);
+ std::string generate_random_file_name();
+ void generate_skip_event(uchar *buf, size_t buf_len, my_off_t pos, THD *thd);
+ /*
+ write gtid_log_event and ignorable_log_event(skip event) to binlog
+ @returns if a problem occurs, false otherwise
+ */
+ bool write_gtid_and_skip_event(my_off_t file_size,
+ group_commit_entry *entry);
+ /*
+ write non-blocking binlog for big transaction
+ @returns true if a problem occurs, false otherwise
+ */
+ bool write_non_blocking_binlog(group_commit_entry *entry);
};
class Log_event_handler
@@ -1476,4 +1500,6 @@ int binlog_commit_by_xid(handlerton *hton, XID *xid);
int binlog_rollback_by_xid(handlerton *hton, XID *xid);
bool write_bin_log_start_alter(THD *thd, bool& partial_alter,
uint64 start_alter_id, bool log_if_exists);
+
+extern mysql_rwlock_t binlog_checksum_rwlock;
#endif /* LOG_H */
diff --git a/sql/mysqld.cc b/sql/mysqld.cc
index dc4a955aa79..8f9bdb02599 100644
--- a/sql/mysqld.cc
+++ b/sql/mysqld.cc
@@ -557,6 +557,7 @@ ulong opt_binlog_commit_wait_usec= 0;
ulong opt_slave_parallel_max_queued= 131072;
my_bool opt_gtid_ignore_duplicates= FALSE;
uint opt_gtid_cleanup_batch_size= 64;
+ulonglong non_blocking_binlog_threshold= ULONGLONG_MAX;
const double log_10[] = {
1e000, 1e001, 1e002, 1e003, 1e004, 1e005, 1e006, 1e007, 1e008, 1e009,
@@ -1059,7 +1060,8 @@ PSI_rwlock_key key_rwlock_LOCK_grant, key_rwlock_LOCK_logger,
key_rwlock_LOCK_vers_stats, key_rwlock_LOCK_stat_serial,
key_rwlock_LOCK_ssl_refresh,
key_rwlock_THD_list,
- key_rwlock_LOCK_all_status_vars;
+ key_rwlock_LOCK_all_status_vars,
+ key_rwlock_binlog_checksum;
static PSI_rwlock_info all_server_rwlocks[]=
{
@@ -1077,7 +1079,8 @@ static PSI_rwlock_info all_server_rwlocks[]=
{ &key_rwlock_LOCK_stat_serial, "TABLE_SHARE::LOCK_stat_serial", 0},
{ &key_rwlock_LOCK_ssl_refresh, "LOCK_ssl_refresh", PSI_FLAG_GLOBAL },
{ &key_rwlock_THD_list, "THD_list::lock", PSI_FLAG_GLOBAL },
- { &key_rwlock_LOCK_all_status_vars, "LOCK_all_status_vars", PSI_FLAG_GLOBAL }
+ { &key_rwlock_LOCK_all_status_vars, "LOCK_all_status_vars", PSI_FLAG_GLOBAL },
+ { &key_rwlock_binlog_checksum, "binlog_checksum_rwlock", PSI_FLAG_GLOBAL }
};
#ifdef HAVE_MMAP
@@ -2105,6 +2108,7 @@ static void clean_up_mutexes()
mysql_mutex_destroy(&LOCK_start_thread);
mysql_mutex_destroy(&LOCK_status);
mysql_rwlock_destroy(&LOCK_all_status_vars);
+ mysql_rwlock_destroy(&binlog_checksum_rwlock);
mysql_mutex_destroy(&LOCK_delayed_insert);
mysql_mutex_destroy(&LOCK_delayed_status);
mysql_mutex_destroy(&LOCK_delayed_create);
@@ -4428,6 +4432,7 @@ static int init_thread_environment()
mysql_rwlock_init(key_rwlock_LOCK_ssl_refresh, &LOCK_ssl_refresh);
mysql_rwlock_init(key_rwlock_LOCK_grant, &LOCK_grant);
mysql_rwlock_init(key_rwlock_LOCK_all_status_vars, &LOCK_all_status_vars);
+ mysql_rwlock_init(key_rwlock_binlog_checksum, &binlog_checksum_rwlock);
mysql_cond_init(key_COND_start_thread, &COND_start_thread, NULL);
#ifdef HAVE_REPLICATION
mysql_mutex_init(key_LOCK_rpl_status, &LOCK_rpl_status, MY_MUTEX_INIT_FAST);
diff --git a/sql/mysqld.h b/sql/mysqld.h
index 4548a247ac1..4f9dfe72b4c 100644
--- a/sql/mysqld.h
+++ b/sql/mysqld.h
@@ -317,6 +317,7 @@ extern const char *encryption_algorithm_names[];
extern long opt_secure_timestamp;
extern uint default_password_lifetime;
extern my_bool disconnect_on_expired_password;
+extern ulonglong non_blocking_binlog_threshold;
enum secure_timestamp { SECTIME_NO, SECTIME_SUPER, SECTIME_REPL, SECTIME_YES };
bool is_set_timestamp_forbidden(THD *thd);
@@ -371,7 +372,8 @@ extern PSI_rwlock_key key_rwlock_LOCK_grant, key_rwlock_LOCK_logger,
key_rwlock_LOCK_system_variables_hash, key_rwlock_query_cache_query_lock,
key_LOCK_SEQUENCE,
key_rwlock_LOCK_vers_stats, key_rwlock_LOCK_stat_serial,
- key_rwlock_THD_list;
+ key_rwlock_THD_list,
+ key_rwlock_binlog_checksum;
#ifdef HAVE_MMAP
extern PSI_cond_key key_PAGE_cond, key_COND_active, key_COND_pool;
diff --git a/sql/sys_vars.cc b/sql/sys_vars.cc
index 47e31291e6b..74e6fe9bb23 100644
--- a/sql/sys_vars.cc
+++ b/sql/sys_vars.cc
@@ -7327,3 +7327,11 @@ static Sys_var_enum Sys_block_encryption_mode(
"AES_ENCRYPT() and AES_DECRYPT() functions",
SESSION_VAR(block_encryption_mode), CMD_LINE(REQUIRED_ARG),
block_encryption_mode_values, DEFAULT(0));
+
+static Sys_var_ulonglong Sys_var_non_blocking_binlog_threshold(
+ "non_blocking_binlog_threshold",
+ "If the binlog size of a transaction exceeds this value, we write it to a "
+ "new temporary file and rename it to the next binlog file.",
+ GLOBAL_VAR(non_blocking_binlog_threshold), CMD_LINE(OPT_ARG),
+ VALID_RANGE(256 * 1024 * 1024, ULONGLONG_MAX), DEFAULT(ULONGLONG_MAX),
+ BLOCK_SIZE(1));
--
2.30.2
1
0
From: Brandon Nesterenko <brandon.nesterenko(a)mariadb.com>
This patch extends the command `SHOW REPLICA HOSTS` with three
columns:
1) Gtid_State_Sent. This represents that latest GTIDs sent to the
replica in each domain. It will always be populated, regardless of
the semi-sync status (i.e. asynchronous connections will still
update this column with the latest GTID state sent to the replica).
2) Gtid_State_Ack. For semi-synchronous connections (only), this
column represents the last GTID in each domain that the replica has
acknowledged.
3) Sync_Status. This value represents the synchronization status of
the replica, and is used to help determine how to interpret the
Gtid_State_Ack column. There are four possible values:
3.1) Initializing. This means the binlog dump thread is still
initializing, and has not yet determined the synchronization status
of the replica.
3.2) Asynchronous: This means the replica is not configured for
semi-sync replication, and thereby, Gtid_State_Ack should always be
empty.
3.3) Semi-sync Stale: This means the replica is configured for
semi-sync replication, however, connected using an old state, and is
not readily able to send ACKs for new transactions. Functionally,
this means that the primary will try to catch the replica up-to-date
by sending transactions which will not be ACKed. Additionally, the
value shown by Gtid_State_Ack will be empty until the replica
catches up and ACKs its first transaction.
3.4) Semi-sync Active: This means the replica is configured for
semi-sync replication, and is readily sending ACKs for new
transactions it receives. It is possible for Gtid_State_Ack to be
empty while Sync_Status is "Semi-sync Active" if no new transactions
have been executed on the primary since the replica has connected.
Additionally, this patch creates a new semantic for the
configuration rpl_semi_sync_master_timeout=0. That is, now when 0,
1) new transactions will not attempt to wait for an ACK before
completing, and 2) the primary will still request ACKs from the
replica for new transactions. This means that Gtid_State_Ack will be
updated for each ACK from the replica and Sync_Status will read as
"Semi-sync Active". Effectively, this creates a mode to mimic the
asynchronous connection behavior, while allowing one to monitor the
progress at which the primary is sending transactions to the replica
via the new columns Gtid_State_Sent and Gtid_State_Ack.
Also note that a new error message was added to account for the case
that Gtid_State_(Sent/Ack) represents a binary log file that was
purged/cannot be found.
The overall implementation is rather simple. It leverages the
existing semi-sync framework, where the replica uses binlog file:pos
to ACK transactions, in order to infer GTID state by performing a
binlog lookup at the time `SHOW REPLICA HOSTS` is executed. In
particular, the Slave_info struct is extended to store 1) the binlog
file:pos pair of the transaction which was last sent to the replica,
2) the binlog file:pos pair that was last ACKed by the replica, and
3) and enum to represent the Sync_Status.
This patch was initially started by @JackSlateur in PR#1427, where
it was then transferred to @an3l who buffed it out in PR#2374, and
final touches were put on by @bnestere.
Reviewed By:
============
<TODO>
---
mysql-test/main/grant_master_admin.result | 2 +-
.../suite/rpl/r/rpl_fail_register.result | 2 +-
.../suite/rpl/r/rpl_mixed_ddl_dml.result | 4 +-
.../suite/rpl/r/rpl_show_slave_hosts.result | 516 ++++++++++-
.../suite/rpl/t/rpl_show_slave_hosts.cnf | 9 +-
.../suite/rpl/t/rpl_show_slave_hosts.test | 808 +++++++++++++++++-
sql/repl_failsafe.cc | 81 +-
sql/semisync_master.cc | 58 +-
sql/semisync_master.h | 93 +-
sql/semisync_master_ack_receiver.cc | 16 +-
sql/share/errmsg-utf8.txt | 2 +
sql/slave.cc | 47 +-
sql/sql_repl.cc | 66 +-
sql/sql_repl.h | 3 +-
14 files changed, 1616 insertions(+), 91 deletions(-)
diff --git a/mysql-test/main/grant_master_admin.result b/mysql-test/main/grant_master_admin.result
index bd08ade940c..97a7b4d0024 100644
--- a/mysql-test/main/grant_master_admin.result
+++ b/mysql-test/main/grant_master_admin.result
@@ -28,7 +28,7 @@ GRANT REPLICATION MASTER ADMIN ON *.* TO `user1`@`localhost`
connect con1,localhost,user1,,;
connection con1;
SHOW SLAVE HOSTS;
-Server_id Host Port Master_id
+Server_id Host Port Master_id Gtid_State_Sent Gtid_State_Ack Sync_Status
connection default;
DROP USER user1@localhost;
#
diff --git a/mysql-test/suite/rpl/r/rpl_fail_register.result b/mysql-test/suite/rpl/r/rpl_fail_register.result
index 0398220c4d0..7af07b335b3 100644
--- a/mysql-test/suite/rpl/r/rpl_fail_register.result
+++ b/mysql-test/suite/rpl/r/rpl_fail_register.result
@@ -14,7 +14,7 @@ set global debug_dbug=@old_dbug;
connection master;
kill DUMP_THREAD;
show slave hosts;
-Server_id Host Port Master_id
+Server_id Host Port Master_id Gtid_State_Sent Gtid_State_Ack Sync_Status
connection slave;
start slave;
include/rpl_end.inc
diff --git a/mysql-test/suite/rpl/r/rpl_mixed_ddl_dml.result b/mysql-test/suite/rpl/r/rpl_mixed_ddl_dml.result
index 0cee79434ee..ec1b8d46824 100644
--- a/mysql-test/suite/rpl/r/rpl_mixed_ddl_dml.result
+++ b/mysql-test/suite/rpl/r/rpl_mixed_ddl_dml.result
@@ -11,8 +11,8 @@ n
2002
connection master;
show slave hosts;
-Server_id Host Port Master_id
-2 127.0.0.1 9999 1
+Server_id Host Port Master_id Gtid_State_Sent Gtid_State_Ack Sync_Status
+2 127.0.0.1 9999 1 0-1-2 Asynchronous
drop table t1;
connection slave;
stop slave;
diff --git a/mysql-test/suite/rpl/r/rpl_show_slave_hosts.result b/mysql-test/suite/rpl/r/rpl_show_slave_hosts.result
index 0c8903378a7..34a5450ed82 100644
--- a/mysql-test/suite/rpl/r/rpl_show_slave_hosts.result
+++ b/mysql-test/suite/rpl/r/rpl_show_slave_hosts.result
@@ -1,20 +1,504 @@
-include/master-slave.inc
-[connection master]
-connect slave2,127.0.0.1,root,,test,$SLAVE_MYPORT2,;
-connection slave2;
-RESET SLAVE;
-CHANGE MASTER TO master_host='127.0.0.1',master_port=MASTER_PORT,master_user='root', master_ssl_verify_server_cert=0;
-START SLAVE IO_THREAD;
-include/wait_for_slave_io_to_start.inc
-connection master;
+include/rpl_init.inc [topology=1->2,1->3]
+connection server_1;
SHOW SLAVE HOSTS;
-Server_id Host Port Master_id
-3 slave2 SLAVE_PORT 1
-2 localhost SLAVE_PORT 1
-connection slave2;
+Server_id Host Port Master_id Gtid_State_Sent Gtid_State_Ack Sync_Status
+2 localhost SLAVE_PORT 1 Asynchronous
+3 slave2 SLAVE2_PORT 1 Asynchronous
+connection server_3;
include/stop_slave_io.inc
-connection master;
+connection server_1;
SHOW SLAVE HOSTS;
-Server_id Host Port Master_id
-2 localhost SLAVE_PORT 1
+Server_id Host Port Master_id Gtid_State_Sent Gtid_State_Ack Sync_Status
+2 localhost SLAVE_PORT 1 Asynchronous
+#
+# MDEV-21322: report slave progress to the primary
+#
+#
+# 21322.0: Test case set-up
+#
+connection server_1;
+set sql_log_bin=0;
+call mtr.add_suppression("Got an error reading communication packets");
+call mtr.add_suppression("Semi-sync master failed on net_flush");
+call mtr.add_suppression("Could not read packet:.* vio_errno: 1158");
+call mtr.add_suppression("Could not write packet:.* vio_errno: 1160");
+set sql_log_bin=1;
+set @save_primary_dbug= @@global.debug_dbug;
+set @save_semisync_timeout= @@global.rpl_semi_sync_master_timeout;
+set @save_semisync_master_enabled= @@global.rpl_semi_sync_master_enabled;
+create table t1 (a int);
+connection server_2;
+set @save_s2_debug= @@GLOBAL.debug_dbug;
+set @save_semisync_server_2_enabled= @@global.rpl_semi_sync_master_enabled;
+set sql_log_bin=0;
+call mtr.add_suppression('Slave I/O: Relay log write failure: could not queue event from master.*');
+call mtr.add_suppression('Slave I/O: Replication event checksum verification failed while reading from network.*');
+call mtr.add_suppression('Replication event checksum verification failed');
+call mtr.add_suppression("Timeout waiting for reply of binlog*");
+call mtr.add_suppression('Found invalid event in binary log');
+call mtr.add_suppression('event read from binlog did not pass crc check');
+call mtr.add_suppression('Event crc check failed! Most likely there is event corruption');
+call mtr.add_suppression('Slave SQL: Error initializing relay log position: I/O error reading event at position .*, error.* 1593');
+call mtr.add_suppression("Semi-sync slave .* reply");
+set sql_log_bin=1;
+connection server_3;
+set @save_s3_debug= @@GLOBAL.debug_dbug;
+set @save_semisync_server_3_enabled= @@global.rpl_semi_sync_master_enabled;
+set sql_log_bin=0;
+call mtr.add_suppression('Slave I/O: Relay log write failure: could not queue event from master.*');
+call mtr.add_suppression('Slave I/O: Replication event checksum verification failed while reading from network.*');
+call mtr.add_suppression('Replication event checksum verification failed');
+call mtr.add_suppression("Timeout waiting for reply of binlog*");
+call mtr.add_suppression('Found invalid event in binary log');
+call mtr.add_suppression('event read from binlog did not pass crc check');
+call mtr.add_suppression('Event crc check failed! Most likely there is event corruption');
+call mtr.add_suppression('Slave SQL: Error initializing relay log position: I/O error reading event at position .*, error.* 1593');
+call mtr.add_suppression("Semi-sync slave .* reply");
+set sql_log_bin=1;
+connection server_1;
+include/save_master_gtid.inc
+connection server_2;
+include/sync_with_master_gtid.inc
+#
+# 21322.1: In a fresh replication state with semi-sync disabled,
+# the Sync_Status column should reflect an asynchronous replication
+# state, and Gtid_State_Sent/Ack should start, and only Gtid_State_Sent
+# should update with new transaction. Note only server_2 is currently
+# connected.
+#
+connection server_1;
+SHOW SLAVE HOSTS;
+Server_id Host Port Master_id Gtid_State_Sent Gtid_State_Ack Sync_Status
+2 localhost SLAVE_PORT 1 0-1-1 Asynchronous
+insert into t1 values (1);
+include/save_master_gtid.inc
+connection server_2;
+include/sync_with_master_gtid.inc
+connection server_1;
+# Gtid_State_Sent should be updated for new transaction
+SHOW SLAVE HOSTS;
+Server_id Host Port Master_id Gtid_State_Sent Gtid_State_Ack Sync_Status
+2 localhost SLAVE_PORT 1 0-1-2 Asynchronous
+#
+# 21322.2: When only the primary enables semi-sync, Sync_Status should
+# still be asynchronous
+#
+connection server_1;
+set global rpl_semi_sync_master_enabled= 1;
+show variables like 'rpl_semi_sync_master_enabled';
+Variable_name Value
+rpl_semi_sync_master_enabled ON
+SHOW SLAVE HOSTS;
+Server_id Host Port Master_id Gtid_State_Sent Gtid_State_Ack Sync_Status
+2 localhost SLAVE_PORT 1 0-1-2 Asynchronous
+#
+# 21322.3: Finalizing the semi-sync connection on server_2 (i.e. by
+# enabling it on the slave) should update Sync_Status to semi-sync
+# active, as the slave is up-to-date.
+#
+connection server_2;
+include/stop_slave.inc
+set global rpl_semi_sync_slave_enabled = 1;
+include/start_slave.inc
+connection server_1;
+SHOW REPLICA HOSTS;
+Server_id Host Port Master_id Gtid_State_Sent Gtid_State_Ack Sync_Status
+2 localhost SLAVE_PORT 1 Semi-sync Active
+#
+# 21322.4: After new semi-sync transactions are ACKed,
+# Gtid_State_Sent/Ack should match gtid_binlog_pos, and Sync_Status
+# should read that semi-sync is active
+#
+connection server_1;
+insert into t1 values (2);
+include/save_master_gtid.inc
+connection server_2;
+include/sync_with_master_gtid.inc
+connection server_1;
+# Ensuring master gtid_binlog_pos matches Gtid_State_Sent
+# Ensuring master gtid_binlog_pos matches Gtid_State_Ack
+# Ensuring Sync_Status is semi-sync active
+SHOW REPLICA HOSTS;
+Server_id Host Port Master_id Gtid_State_Sent Gtid_State_Ack Sync_Status
+2 localhost SLAVE_PORT 1 0-1-3 0-1-3 Semi-sync Active
+#
+# 21322.5: When connecting a new slave (server_id 3) which initially has
+# semi-sync disabled, SHOW SLAVE HOSTS on the master should show its
+# Sync_Status as asynchronous (while server_id 2 is still semi-sync
+# active).
+#
+connection server_3;
+include/start_slave.inc
+include/sync_with_master_gtid.inc
+connection server_1;
+SHOW REPLICA HOSTS;
+Server_id Host Port Master_id Gtid_State_Sent Gtid_State_Ack Sync_Status
+3 slave2 SLAVE2_PORT 1 0-1-3 Asynchronous
+2 localhost SLAVE_PORT 1 0-1-3 0-1-3 Semi-sync Active
+#
+# 21322.6: Reconnecting server_3 as a semi-sync enabled replica should
+# result in a Sync_Status reflecting active semi-sync
+#
+connection server_3;
+include/stop_slave.inc
+set global rpl_semi_sync_slave_enabled = 1;
+include/start_slave.inc
+connection server_1;
+show status like 'Rpl_semi_sync_master_clients';
+Variable_name Value
+Rpl_semi_sync_master_clients 2
+SHOW REPLICA HOSTS;
+Server_id Host Port Master_id Gtid_State_Sent Gtid_State_Ack Sync_Status
+3 slave2 SLAVE2_PORT 1 Semi-sync Active
+2 localhost SLAVE_PORT 1 0-1-3 0-1-3 Semi-sync Active
+#
+# 21322.7: New transactions on the master should update Gtid_State_Sent
+# when sent to the slave, and Gtid_State_Ack once receiving an ACK
+#
+connection server_2;
+connection server_1;
+SET @@GLOBAL.debug_dbug="+d,pause_ack_thread_on_next_ack";
+connection default;
+insert into t1 values (3);
+connection server_1;
+# waiting for pause_ack_reply_to_binlog
+SET debug_sync='now WAIT_FOR pause_ack_reply_to_binlog';
+# Waiting for Gtid_State_Sent to reflect latest transaction for all replicas..
+# Ensuring Gtid_State_Ack is not yet updated (as ACK thread is paused)
+SHOW REPLICA HOSTS;
+Server_id Host Port Master_id Gtid_State_Sent Gtid_State_Ack Sync_Status
+3 slave2 SLAVE2_PORT 1 0-1-4 Semi-sync Active
+2 localhost SLAVE_PORT 1 0-1-4 0-1-3 Semi-sync Active
+connection server_1;
+SET debug_sync='now SIGNAL resume_ack_thread';
+connection default;
+connection server_1;
+# Waiting for Gtid_State_Ack to reflect latest transaction for all replicas..
+SHOW REPLICA HOSTS;
+Server_id Host Port Master_id Gtid_State_Sent Gtid_State_Ack Sync_Status
+3 slave2 SLAVE2_PORT 1 0-1-4 0-1-4 Semi-sync Active
+2 localhost SLAVE_PORT 1 0-1-4 0-1-4 Semi-sync Active
+# Reset debug state
+SET @@GLOBAL.debug_dbug= @save_primary_dbug;
+SET debug_sync='RESET';
+#
+# 21322.8: Holding one replica's ACK (server_2) should result in
+# Gtid_State_Ack of server_3 updating to the most recent GTID, while
+# server_id 2 has the old GTID. Note that we need to use debug_sync to
+# synchronize the ACKs of both server_2 and server_3, so server_3 can't
+# ACK the transaction before server_2's binlog dump thread sends the
+# transaction (which would negate the need for server_2 to ACK at all,
+# resulting in MTR hanging on its expected debug_sync WAIT_FOR point.)
+#
+connection server_2;
+include/stop_slave.inc
+set @@GLOBAL.debug_dbug="+d,synchronize_semisync_slave_reply";
+include/start_slave.inc
+connection server_3;
+include/stop_slave.inc
+set @@GLOBAL.debug_dbug="+d,synchronize_semisync_slave_reply";
+include/start_slave.inc
+connection server_1;
+# Waiting for master to recognize slave restart..
+insert into t1 values (4);
+connection server_2;
+set debug_sync= "now WAIT_FOR at_slave_reply";
+connection server_3;
+set debug_sync= "now WAIT_FOR at_slave_reply";
+connection default;
+# Ensure Gtid_State_Sent reflects latest transaction (0-1-5) for all replicas..
+connection server_3;
+set debug_sync= "now SIGNAL reply_ack_to_master";
+connection server_1;
+# Wait for Gtid_State_Ack to show the latest transaction for server_3..
+# Only server_3 should have ACKed the new GTID, server_2 should not due to debug_sync holding off the ACK
+SHOW REPLICA HOSTS;
+Server_id Host Port Master_id Gtid_State_Sent Gtid_State_Ack Sync_Status
+3 slave2 SLAVE2_PORT 1 0-1-5 0-1-5 Semi-sync Active
+2 localhost SLAVE_PORT 1 0-1-5 Semi-sync Active
+connection server_2;
+# Resume slave so it can ACK the transaction
+set debug_sync= "now SIGNAL reply_ack_to_master";
+# Waiting for Gtid_State_Ack to reflect latest transaction for all replicas..
+connection server_1;
+SHOW REPLICA HOSTS;
+Server_id Host Port Master_id Gtid_State_Sent Gtid_State_Ack Sync_Status
+3 slave2 SLAVE2_PORT 1 0-1-5 0-1-5 Semi-sync Active
+2 localhost SLAVE_PORT 1 0-1-5 0-1-5 Semi-sync Active
+connection server_2;
+include/stop_slave.inc
+set @@GLOBAL.debug_dbug= @save_s2_debug;
+SET debug_sync='RESET';
+include/start_slave.inc
+connection server_3;
+include/stop_slave.inc
+set @@GLOBAL.debug_dbug= @save_s3_debug;
+SET debug_sync='RESET';
+include/start_slave.inc
+connection server_1;
+include/save_master_gtid.inc
+connection server_2;
+include/sync_with_master_gtid.inc
+connection server_3;
+include/sync_with_master_gtid.inc
+#
+# 21322.9: If a server is behind when connecting to a primary (i.e. the
+# primary has newer transactions), the replica Sync_Status should
+# present as semi-sync stale and the Gtid_State_Ack should not populate
+# until it has reached Semi-Sync Active
+#
+connection server_2;
+include/stop_slave.inc
+connection server_1;
+insert into t1 values (5);
+include/save_master_gtid.inc
+# Pause dump_thread of server_2 (server_3 won't be affected as it has
+# already successfully ACKed the new transaction)
+set @@global.debug_dbug= "+d,pause_dump_thread_after_sending_next_full_trx";
+connection server_2;
+include/start_slave.inc
+connection server_1;
+set debug_sync= 'now WAIT_FOR dump_thread_paused';
+# Ensure Gtid_State_Sent is updated to represent new transaction has
+# been sent to both replicas
+# Ensure Sync_Status is Semi-sync Stale for the debug_sync held dump
+# thread, as it hasn't yet got "up-to-date"
+set debug_sync= 'now SIGNAL dump_thread_continue';
+# Ensure Sync_Status will automatically update to Semi-sync Active
+# once the last stale transaction has finished sending
+set @@global.debug_dbug= @save_primary_dbug;
+SET debug_sync='RESET';
+include/save_master_gtid.inc
+connection server_2;
+include/sync_with_master_gtid.inc
+connection server_3;
+include/sync_with_master_gtid.inc
+#
+# 21322.10a: If one replica errors (i.e. server_2 by injecting
+# corrupt_gtid_event with debug_dbug), it shouldn't send its ACK,
+# and server_3 does ACK; then Gtid_State_Sent should still reflect the
+# new transaction for each replica, but Gtid_State_Ack should only be
+# updated by the successful transaction. When the errored replica
+# reconnects, it shouldn't ACK the transaction, so its on-reconnect
+# Gtid_State_Ack value should be empty, but still have a Sync_Status
+# of "Semi-sync Active" when it receives the latest transaction, as it
+# will be ready to ACK new transactions going forward.
+#
+connection server_2;
+include/stop_slave.inc
+SET @@GLOBAL.debug_dbug= "+d,corrupt_gtid_event";
+include/start_slave.inc
+# Set-up server_3 for 10b (so we don't have to restart the slave)
+connection server_3;
+include/stop_slave.inc
+set @@GLOBAL.debug_dbug="+d,synchronize_semisync_slave_reply";
+include/start_slave.inc
+connection server_1;
+# Waiting for master to recognize slave restarts..
+connection server_1;
+insert into t1 values (6);
+# Debug_sync is irrelevant to this testcase (10a) but we must do it to
+# allow server_3 to ACK now (Note debug_sync is needed for 10b)
+connection server_3;
+set debug_sync= "now WAIT_FOR at_slave_reply";
+set debug_sync= "now SIGNAL reply_ack_to_master";
+connection server_1;
+include/save_master_gtid.inc
+# Waiting for Gtid_State_Sent to reflect latest transaction for all replicas..
+connection server_2;
+include/wait_for_slave_io_error.inc [errno=1595]
+set @@GLOBAL.debug_dbug= @save_s2_debug;
+connection server_1;
+# Only server_3 should ACKed have the new GTID, server_2 should not due to corrupt_queue_event
+SHOW REPLICA HOSTS;
+Server_id Host Port Master_id Gtid_State_Sent Gtid_State_Ack Sync_Status
+3 slave2 SLAVE2_PORT 1 0-1-7 0-1-7 Semi-sync Active
+2 localhost SLAVE_PORT 1 0-1-7 Semi-sync Active
+connection server_2;
+include/start_slave.inc
+include/sync_with_master_gtid.inc
+connection server_1;
+# With replica restarted/synced, its Gtid_State_Ack should be empty with Sync_Status semi-sync active
+SHOW REPLICA HOSTS;
+Server_id Host Port Master_id Gtid_State_Sent Gtid_State_Ack Sync_Status
+2 localhost SLAVE_PORT 1 0-1-7 Semi-sync Active
+3 slave2 SLAVE2_PORT 1 0-1-7 0-1-7 Semi-sync Active
+#
+# 21322.10b: Succeeding the previous 10a test, if server_3 now stalls
+# (i.e. using debug_sync), then the previously errored server_2 should
+# receive and ACK new transactions as a "lone" replica, and update its
+# Gtid_State_* columns appropriately, whereas server_3's Gtid_State_Ack
+# column should not be updated.
+#
+insert into t1 values (7);
+include/save_master_gtid.inc
+connection server_2;
+include/sync_with_master_gtid.inc
+# Waiting for Gtid_State_Sent to reflect latest transaction for all replicas..
+connection server_3;
+set debug_sync= "now WAIT_FOR at_slave_reply";
+connection server_1;
+# Only server_2 should have ACKed the new GTID ACKed, server_3 should not due to stall
+SHOW REPLICA HOSTS;
+Server_id Host Port Master_id Gtid_State_Sent Gtid_State_Ack Sync_Status
+2 localhost SLAVE_PORT 1 0-1-8 0-1-8 Semi-sync Active
+3 slave2 SLAVE2_PORT 1 0-1-8 0-1-7 Semi-sync Active
+# Resume server_3
+connection server_3;
+set debug_sync= "now SIGNAL reply_ack_to_master";
+include/sync_with_master_gtid.inc
+SET debug_sync='RESET';
+include/stop_slave.inc
+set @@GLOBAL.debug_dbug= @save_s3_debug;
+SET debug_sync='RESET';
+include/start_slave.inc
+#
+# 21322.11: Configuration rpl_semi_sync_master_timeout of 0 should
+# have transaction behavior match asynchronous behavior (i.e. trxs don't
+# need to wait for ACKs), yet the slave should still send ACKs as a
+# normal semi-sync replica, and Gtid_State_Ack should still be updated
+# accordingly.
+#
+# The actual behavior tested in this case is as follows:
+# a) Transactions won't await ACKs to complete
+# b) Semi-sync remains ON when a transaction completes without an ACK
+# c) Gtid_State_Ack is updated accordingly for each replica's ACK (even
+# when it is behind). Here, we hold both replicas using DEBUG_SYNC to
+# not send their ACKs, meanwhile, we continue creating transactions
+# on the primary.
+# d) If a "very lagged" replica sends an ACK for a transaction from a
+# purged binlog, the Gtid_State_Ack value should be cleared, and
+# issue a warning to the user with the slave's last ACKed binlog
+# coordinate (i.e. filename and position).
+#
+connection server_1;
+set global rpl_semi_sync_master_timeout=0;
+connection server_2;
+include/stop_slave.inc
+SET @@GLOBAL.DEBUG_DBUG="+d,synchronize_semisync_slave_reply";
+include/start_slave.inc
+connection server_3;
+include/stop_slave.inc
+SET @@GLOBAL.DEBUG_DBUG="+d,synchronize_semisync_slave_reply";
+include/start_slave.inc
+connection server_1;
+# Waiting for master to recognize slave restarts..
+#
+# 21322.11.a
+connection server_1;
+insert into t1 values (8);
+# Waiting for Gtid_State_Sent to reflect latest transaction for all replicas..
+connection server_2;
+SET debug_sync='now WAIT_FOR at_slave_reply';
+connection server_3;
+SET debug_sync='now WAIT_FOR at_slave_reply';
+# Gtid_State_Ack should be empty for both replicas (as they were restarted)..
+#
+# 21322.11.b
+connection server_1;
+# Ensuring semi-sync status on primary is correct..
+#
+# 21322.11.c
+connection server_1;
+insert into t1 values (9);
+include/save_master_gtid.inc
+# Waiting for Gtid_State_Sent to reflect latest transaction for all replicas..
+connection server_1;
+# server_2 and 3 should both show an empty ACK state
+SHOW REPLICA HOSTS;
+Server_id Host Port Master_id Gtid_State_Sent Gtid_State_Ack Sync_Status
+3 slave2 SLAVE2_PORT 1 TRX2_GTID Semi-sync Active
+2 localhost SLAVE_PORT 1 TRX2_GTID Semi-sync Active
+# Let server_2 ACK just the first transaction
+connection server_2;
+set debug_sync= "now SIGNAL reply_ack_to_master";
+set debug_sync= "now WAIT_FOR at_slave_reply";
+connection server_1;
+# Waiting for server_2 Gtid_State_Ack to reflect first transaction
+# Let server_2 ACK the second transaction
+connection server_2;
+set debug_sync= "now SIGNAL reply_ack_to_master";
+include/sync_with_master_gtid.inc
+connection server_1;
+# Waiting for server_2 Gtid_State_Ack to reflect second transaction
+# Let server_3 now ACK the first transaction
+connection server_3;
+set debug_sync= "now SIGNAL reply_ack_to_master";
+set debug_sync= "now WAIT_FOR at_slave_reply";
+connection server_1;
+# Waiting for server_3 Gtid_State_Ack to reflect first transaction
+# Let server_3 ACK the second transaction
+connection server_3;
+set debug_sync= "now SIGNAL reply_ack_to_master";
+include/sync_with_master_gtid.inc
+connection server_1;
+# Waiting for Gtid_State_Ack to reflect second transaction for both servers
+#
+# 21322.11.d
+connection server_1;
+FLUSH LOGS;
+insert into t1 values (10);
+include/save_master_gtid.inc
+# Waiting for Gtid_State_Sent to reflect latest transaction for all replicas..
+connection server_2;
+set debug_sync= "now WAIT_FOR at_slave_reply";
+connection server_3;
+set debug_sync= "now WAIT_FOR at_slave_reply";
+connection server_1;
+# server_2 and 3 should both show ACKed TRX2 (with TRX3 Sent)
+SHOW REPLICA HOSTS;
+Server_id Host Port Master_id Gtid_State_Sent Gtid_State_Ack Sync_Status
+3 slave2 SLAVE2_PORT 1 TRX3_GTID TRX2_GTID Semi-sync Active
+2 localhost SLAVE_PORT 1 TRX3_GTID TRX2_GTID Semi-sync Active
+include/wait_for_purge.inc "master-bin.000002"
+# Master should warn that the binary log which contains the last ACKed
+# binlog coordinates has been purged, and clear Gtid_State_Ack
+SHOW REPLICA HOSTS;
+Server_id Host Port Master_id Gtid_State_Sent Gtid_State_Ack Sync_Status
+3 slave2 SLAVE2_PORT 1 TRX3_GTID Semi-sync Active
+2 localhost SLAVE_PORT 1 TRX3_GTID Semi-sync Active
+Warnings:
+Warning 4200 Error constructing GTID state for binlog position TRX2_BINLOG_POS in file 'TRX2_BINLOG_FILE': Could not find binary log file. Probably the slave state is too old and required binlog files have been purged.
+Warning 4200 Error constructing GTID state for binlog position TRX2_BINLOG_POS in file 'TRX2_BINLOG_FILE': Could not find binary log file. Probably the slave state is too old and required binlog files have been purged.
+# Let servers ACK new transaction
+connection server_2;
+set debug_sync= "now SIGNAL reply_ack_to_master";
+connection server_3;
+set debug_sync= "now SIGNAL reply_ack_to_master";
+connection server_1;
+# Waiting for Gtid_State_Ack to reflect latest transaction for all replicas..
+# Gtid_State_Ack should now show the latest transaction GTID
+SHOW REPLICA HOSTS;
+Server_id Host Port Master_id Gtid_State_Sent Gtid_State_Ack Sync_Status
+3 slave2 SLAVE2_PORT 1 TRX3_GTID TRX3_GTID Semi-sync Active
+2 localhost SLAVE_PORT 1 TRX3_GTID TRX3_GTID Semi-sync Active
+connection server_2;
+include/sync_with_master_gtid.inc
+connection server_3;
+include/sync_with_master_gtid.inc
+#
+# MDEV-21322 Cleanup
+connection server_1;
+set @@global.debug_dbug= @save_primary_dbug;
+set @@global.rpl_semi_sync_master_timeout= @save_semisync_timeout;
+set @@global.rpl_semi_sync_master_enabled= @save_semisync_master_enabled;
+drop table t1;
+include/save_master_gtid.inc
+connection server_2;
+include/sync_with_master_gtid.inc
+include/stop_slave.inc
+set @@global.rpl_semi_sync_slave_enabled= @save_semisync_server_2_enabled;
+SET @@GLOBAL.debug_dbug= "";
+SET debug_sync='RESET';
+include/start_slave.inc
+connection server_3;
+include/sync_with_master_gtid.inc
+include/stop_slave.inc
+set @@global.rpl_semi_sync_slave_enabled= @save_semisync_server_3_enabled;
+SET @@GLOBAL.debug_dbug= "";
+SET debug_sync='RESET';
+include/start_slave.inc
+#
+# End of MDEV-21322 tests
+#
include/rpl_end.inc
diff --git a/mysql-test/suite/rpl/t/rpl_show_slave_hosts.cnf b/mysql-test/suite/rpl/t/rpl_show_slave_hosts.cnf
index 288f0132fba..43f9f1a294d 100644
--- a/mysql-test/suite/rpl/t/rpl_show_slave_hosts.cnf
+++ b/mysql-test/suite/rpl/t/rpl_show_slave_hosts.cnf
@@ -2,19 +2,24 @@
[mysqld.1]
server_id=1
+log_warnings=9
[mysqld.2]
server_id=2
report-host=
report-user=
+log_slave_updates=1
[mysqld.3]
server_id=3
report-host=slave2
slave-net-timeout=5
+log_slave_updates=1
+log_bin=slave2
[ENV]
-SLAVE_MYPORT2= @mysqld.3.port
-SLAVE_MYSOCK2= @mysqld.3.socket
+SERVER_MYPORT_1= @mysqld.1.port
+SERVER_MYPORT_2= @mysqld.2.port
+SERVER_MYPORT_3= @mysqld.3.port
diff --git a/mysql-test/suite/rpl/t/rpl_show_slave_hosts.test b/mysql-test/suite/rpl/t/rpl_show_slave_hosts.test
index afac298495f..a63c1662b81 100644
--- a/mysql-test/suite/rpl/t/rpl_show_slave_hosts.test
+++ b/mysql-test/suite/rpl/t/rpl_show_slave_hosts.test
@@ -9,17 +9,10 @@
# Remove the "Rpl_recovery_rank" column from SHOW SLAVE HOSTS, It is not
# implemented.
#######################################################################
-source include/master-slave.inc;
-connect (slave2,127.0.0.1,root,,test,$SLAVE_MYPORT2,);
+--let $rpl_topology= 1->2,1->3
+--source include/rpl_init.inc
-connection slave2;
-RESET SLAVE;
---replace_result $MASTER_MYPORT MASTER_PORT
---eval CHANGE MASTER TO master_host='127.0.0.1',master_port=$MASTER_MYPORT,master_user='root', master_ssl_verify_server_cert=0
-START SLAVE IO_THREAD;
-source include/wait_for_slave_io_to_start.inc;
-
-connection master;
+connection server_1;
let $show_statement= SHOW SLAVE HOSTS;
let $field= Server_id;
# 3 is server_id of slave2.
@@ -30,14 +23,13 @@ source include/wait_show_condition.inc;
# HOSTS, when that slave is much slower to register due to thread scheduling.
let $condition= ='2';
source include/wait_show_condition.inc;
---replace_column 3 'SLAVE_PORT'
---replace_result $SLAVE_MYPORT SLAVE_PORT $DEFAULT_MASTER_PORT DEFAULT_PORT
+--replace_result $SLAVE_MYPORT SLAVE_PORT $SERVER_MYPORT_3 SLAVE2_PORT $DEFAULT_MASTER_PORT DEFAULT_PORT
SHOW SLAVE HOSTS;
-connection slave2;
+connection server_3;
--source include/stop_slave_io.inc
-connection master;
+connection server_1;
let $show_statement= SHOW SLAVE HOSTS;
let $field= Server_id;
# 3 is server_id of slave2.
@@ -49,4 +41,792 @@ source include/wait_show_condition.inc;
--replace_result $SLAVE_MYPORT SLAVE_PORT
SHOW SLAVE HOSTS;
+
+--echo #
+--echo # MDEV-21322: report slave progress to the primary
+--echo #
+
+--echo #
+--echo # 21322.0: Test case set-up
+--echo #
+connection server_1;
+set sql_log_bin=0;
+call mtr.add_suppression("Got an error reading communication packets");
+call mtr.add_suppression("Semi-sync master failed on net_flush");
+call mtr.add_suppression("Could not read packet:.* vio_errno: 1158");
+call mtr.add_suppression("Could not write packet:.* vio_errno: 1160");
+set sql_log_bin=1;
+set @save_primary_dbug= @@global.debug_dbug;
+set @save_semisync_timeout= @@global.rpl_semi_sync_master_timeout;
+set @save_semisync_master_enabled= @@global.rpl_semi_sync_master_enabled;
+create table t1 (a int);
+
+connection server_2;
+set @save_s2_debug= @@GLOBAL.debug_dbug;
+set @save_semisync_server_2_enabled= @@global.rpl_semi_sync_master_enabled;
+set sql_log_bin=0;
+call mtr.add_suppression('Slave I/O: Relay log write failure: could not queue event from master.*');
+call mtr.add_suppression('Slave I/O: Replication event checksum verification failed while reading from network.*');
+call mtr.add_suppression('Replication event checksum verification failed');
+call mtr.add_suppression("Timeout waiting for reply of binlog*");
+call mtr.add_suppression('Found invalid event in binary log');
+call mtr.add_suppression('event read from binlog did not pass crc check');
+call mtr.add_suppression('Event crc check failed! Most likely there is event corruption');
+call mtr.add_suppression('Slave SQL: Error initializing relay log position: I/O error reading event at position .*, error.* 1593');
+call mtr.add_suppression("Semi-sync slave .* reply");
+set sql_log_bin=1;
+
+connection server_3;
+set @save_s3_debug= @@GLOBAL.debug_dbug;
+set @save_semisync_server_3_enabled= @@global.rpl_semi_sync_master_enabled;
+set sql_log_bin=0;
+call mtr.add_suppression('Slave I/O: Relay log write failure: could not queue event from master.*');
+call mtr.add_suppression('Slave I/O: Replication event checksum verification failed while reading from network.*');
+call mtr.add_suppression('Replication event checksum verification failed');
+call mtr.add_suppression("Timeout waiting for reply of binlog*");
+call mtr.add_suppression('Found invalid event in binary log');
+call mtr.add_suppression('event read from binlog did not pass crc check');
+call mtr.add_suppression('Event crc check failed! Most likely there is event corruption');
+call mtr.add_suppression('Slave SQL: Error initializing relay log position: I/O error reading event at position .*, error.* 1593');
+call mtr.add_suppression("Semi-sync slave .* reply");
+set sql_log_bin=1;
+
+--connection server_1
+--source include/save_master_gtid.inc
+--connection server_2
+--source include/sync_with_master_gtid.inc
+
+
+--echo #
+--echo # 21322.1: In a fresh replication state with semi-sync disabled,
+--echo # the Sync_Status column should reflect an asynchronous replication
+--echo # state, and Gtid_State_Sent/Ack should start, and only Gtid_State_Sent
+--echo # should update with new transaction. Note only server_2 is currently
+--echo # connected.
+--echo #
+--connection server_1
+--replace_result $SLAVE_MYPORT SLAVE_PORT $DEFAULT_MASTER_PORT DEFAULT_PORT
+SHOW SLAVE HOSTS;
+insert into t1 values (1);
+--source include/save_master_gtid.inc
+--connection server_2
+--source include/sync_with_master_gtid.inc
+
+--connection server_1
+--echo # Gtid_State_Sent should be updated for new transaction
+--replace_result $SLAVE_MYPORT SLAVE_PORT $DEFAULT_MASTER_PORT DEFAULT_PORT
+SHOW SLAVE HOSTS;
+--let $master_gtid= `SELECT @@gtid_binlog_pos`
+--let $gtid_sent= query_get_value(show slave hosts, Gtid_State_Sent, 1)
+if (`SELECT strcmp("$master_gtid","$gtid_sent") != 0`)
+{
+ --echo # Master gtid_binlog_pos: $master_gtid
+ --echo # Gtid_State_Sent: $gtid_sent
+ --die Master did not update Gtid_State_Sent for asynchronous replica
+}
+
+
+--echo #
+--echo # 21322.2: When only the primary enables semi-sync, Sync_Status should
+--echo # still be asynchronous
+--echo #
+--connection server_1
+set global rpl_semi_sync_master_enabled= 1;
+show variables like 'rpl_semi_sync_master_enabled';
+--replace_result $SLAVE_MYPORT SLAVE_PORT $DEFAULT_MASTER_PORT DEFAULT_PORT
+SHOW SLAVE HOSTS;
+
+
+--echo #
+--echo # 21322.3: Finalizing the semi-sync connection on server_2 (i.e. by
+--echo # enabling it on the slave) should update Sync_Status to semi-sync
+--echo # active, as the slave is up-to-date.
+--echo #
+
+--connection server_2
+--source include/stop_slave.inc
+set global rpl_semi_sync_slave_enabled = 1;
+--source include/start_slave.inc
+
+--connection server_1
+let $status_var= Rpl_semi_sync_master_clients;
+let $status_var_value= 1;
+source include/wait_for_status_var.inc;
+
+--replace_result $SLAVE_MYPORT SLAVE_PORT $SERVER_MYPORT_3 SLAVE2_PORT $DEFAULT_MASTER_PORT DEFAULT_PORT
+SHOW REPLICA HOSTS;
+
+
+--echo #
+--echo # 21322.4: After new semi-sync transactions are ACKed,
+--echo # Gtid_State_Sent/Ack should match gtid_binlog_pos, and Sync_Status
+--echo # should read that semi-sync is active
+--echo #
+--connection server_1
+--let $nextval= `SELECT max(a)+1 from t1`
+--eval insert into t1 values ($nextval)
+--source include/save_master_gtid.inc
+--connection server_2
+--source include/sync_with_master_gtid.inc
+
+--connection server_1
+--let $master_gtid= `SELECT @@gtid_binlog_pos`
+
+--echo # Ensuring master gtid_binlog_pos matches Gtid_State_Sent
+--let $gtid_sent= query_get_value(show slave hosts, Gtid_State_Sent, 1)
+if (`SELECT strcmp("$master_gtid","$gtid_sent") != 0`)
+{
+ --echo # Master gtid_binlog_pos: $master_gtid
+ --echo # Gtid_State_Sent: $gtid_sent
+ --die Master's gtid_binlog_pos should match Gtid_State_Sent, but doesn't
+}
+
+--echo # Ensuring master gtid_binlog_pos matches Gtid_State_Ack
+--let $gtid_ack= query_get_value(show slave hosts, Gtid_State_Ack, 1)
+if (`SELECT strcmp("$master_gtid","$gtid_ack") != 0`)
+{
+ --echo # Master gtid_binlog_pos: $master_gtid
+ --echo # Gtid_State_Ack: $gtid_ack
+ --die Master's gtid_binlog_pos should match Gtid_State_Ack, but doesn't
+}
+
+--echo # Ensuring Sync_Status is semi-sync active
+--let $sync_status= query_get_value(show slave hosts, Sync_Status, 1)
+if (`SELECT strcmp("$sync_status","semi-sync active") != 0`)
+{
+ --echo # Sync_Status: $sync_status
+ --die Incorrect value for Sync_Status, should be "semi-sync active"
+}
+
+--replace_result $SLAVE_MYPORT SLAVE_PORT $SERVER_MYPORT_3 SLAVE2_PORT $DEFAULT_MASTER_PORT DEFAULT_PORT
+SHOW REPLICA HOSTS;
+
+
+--echo #
+--echo # 21322.5: When connecting a new slave (server_id 3) which initially has
+--echo # semi-sync disabled, SHOW SLAVE HOSTS on the master should show its
+--echo # Sync_Status as asynchronous (while server_id 2 is still semi-sync
+--echo # active).
+--echo #
+# Iniital replication state on server_3 is off
+connection server_3;
+--source include/start_slave.inc
+--source include/sync_with_master_gtid.inc
+
+connection server_1;
+--replace_result $SLAVE_MYPORT SLAVE_PORT $SERVER_MYPORT_3 SLAVE2_PORT $DEFAULT_MASTER_PORT DEFAULT_PORT
+SHOW REPLICA HOSTS;
+
+--echo #
+--echo # 21322.6: Reconnecting server_3 as a semi-sync enabled replica should
+--echo # result in a Sync_Status reflecting active semi-sync
+--echo #
+
+connection server_3;
+--source include/stop_slave.inc
+set global rpl_semi_sync_slave_enabled = 1;
+--source include/start_slave.inc
+
+connection server_1;
+let $status_var= Rpl_semi_sync_master_clients;
+let $status_var_value= 2;
+source include/wait_for_status_var.inc;
+show status like 'Rpl_semi_sync_master_clients';
+--replace_result $SLAVE_MYPORT SLAVE_PORT $SERVER_MYPORT_3 SLAVE2_PORT $DEFAULT_MASTER_PORT DEFAULT_PORT
+SHOW REPLICA HOSTS;
+
+
+--echo #
+--echo # 21322.7: New transactions on the master should update Gtid_State_Sent
+--echo # when sent to the slave, and Gtid_State_Ack once receiving an ACK
+--echo #
+
+--connection server_2
+let $server_2_sent_ack= query_get_value(show status like 'Rpl_semi_sync_slave_send_ack', Value, 1);
+
+--connection server_1
+SET @@GLOBAL.debug_dbug="+d,pause_ack_thread_on_next_ack";
+
+# Write the new event
+--connection default
+--let $old_binlog_gtid= `SELECT @@gtid_binlog_pos`
+--let $nextval= `SELECT max(a)+1 from t1`
+--send_eval insert into t1 values ($nextval)
+
+--connection server_1
+--echo # waiting for pause_ack_reply_to_binlog
+SET debug_sync='now WAIT_FOR pause_ack_reply_to_binlog';
+--let $new_binlog_gtid= `SELECT @@gtid_binlog_pos`
+
+--echo # Waiting for Gtid_State_Sent to reflect latest transaction for all replicas..
+let $show_statement= SHOW REPLICA HOSTS;
+let $field= Gtid_State_Sent;
+let $condition= LIKE '$new_binlog_gtid';
+let $wait_for_all= 1;
+source include/wait_show_condition.inc;
+
+--echo # Ensuring Gtid_State_Ack is not yet updated (as ACK thread is paused)
+--let $gtid_ack= query_get_value(show slave hosts, Gtid_State_Ack, 1)
+if (`SELECT strcmp("$old_master_gtid","$gtid_ack") != 0`)
+{
+ --echo # Master gtid_binlog_pos: $master_gtid
+ --echo # Gtid_State_Ack: $gtid_ack
+ --die Gtid_State_Ack should not yet reflect the GTID of the new transaction
+}
+
+--replace_result $SLAVE_MYPORT SLAVE_PORT $SERVER_MYPORT_3 SLAVE2_PORT $DEFAULT_MASTER_PORT DEFAULT_PORT
+SHOW REPLICA HOSTS;
+
+--connection server_1
+SET debug_sync='now SIGNAL resume_ack_thread';
+
+--connection default
+--reap
+--connection server_1
+
+--echo # Waiting for Gtid_State_Ack to reflect latest transaction for all replicas..
+let $show_statement= SHOW REPLICA HOSTS;
+let $field= Gtid_State_Ack;
+let $condition= LIKE '$new_binlog_gtid';
+let $wait_for_all= 1;
+source include/wait_show_condition.inc;
+
+--replace_result $SLAVE_MYPORT SLAVE_PORT $SERVER_MYPORT_3 SLAVE2_PORT $DEFAULT_MASTER_PORT DEFAULT_PORT
+SHOW REPLICA HOSTS;
+
+--echo # Reset debug state
+SET @@GLOBAL.debug_dbug= @save_primary_dbug;
+SET debug_sync='RESET';
+
+
+--echo #
+--echo # 21322.8: Holding one replica's ACK (server_2) should result in
+--echo # Gtid_State_Ack of server_3 updating to the most recent GTID, while
+--echo # server_id 2 has the old GTID. Note that we need to use debug_sync to
+--echo # synchronize the ACKs of both server_2 and server_3, so server_3 can't
+--echo # ACK the transaction before server_2's binlog dump thread sends the
+--echo # transaction (which would negate the need for server_2 to ACK at all,
+--echo # resulting in MTR hanging on its expected debug_sync WAIT_FOR point.)
+--echo #
+
+--connection server_2
+--source include/stop_slave.inc
+set @@GLOBAL.debug_dbug="+d,synchronize_semisync_slave_reply";
+--source include/start_slave.inc
+--connection server_3
+--source include/stop_slave.inc
+set @@GLOBAL.debug_dbug="+d,synchronize_semisync_slave_reply";
+--source include/start_slave.inc
+
+--connection server_1
+--echo # Waiting for master to recognize slave restart..
+let $status_var= Rpl_semi_sync_master_clients;
+let $status_var_value= 2;
+source include/wait_for_status_var.inc;
+
+--let $nextval= `SELECT max(a)+1 from t1`
+--send_eval insert into t1 values ($nextval)
+
+--connection server_2
+set debug_sync= "now WAIT_FOR at_slave_reply";
+--connection server_3
+set debug_sync= "now WAIT_FOR at_slave_reply";
+
+# New transaction is binlogged at this point, so we can query gtid_binlog_pos
+--connection default
+--let $binlog_gtid= `SELECT @@gtid_binlog_pos`
+
+--echo # Ensure Gtid_State_Sent reflects latest transaction ($binlog_gtid) for all replicas..
+let $show_statement= SHOW REPLICA HOSTS;
+let $field= Gtid_State_Sent;
+let $condition= LIKE '$binlog_gtid';
+let $wait_for_all= 1;
+source include/wait_show_condition.inc;
+
+--connection server_3
+set debug_sync= "now SIGNAL reply_ack_to_master";
+
+--connection server_1
+--reap
+
+--echo # Wait for Gtid_State_Ack to show the latest transaction for server_3..
+let $show_statement= SHOW REPLICA HOSTS;
+let $field= Gtid_State_Ack;
+let $condition= LIKE '$binlog_gtid';
+let $wait_for_all= 0;
+source include/wait_show_condition.inc;
+
+--echo # Only server_3 should have ACKed the new GTID, server_2 should not due to debug_sync holding off the ACK
+--replace_result $SLAVE_MYPORT SLAVE_PORT $SERVER_MYPORT_3 SLAVE2_PORT $DEFAULT_MASTER_PORT DEFAULT_PORT
+SHOW REPLICA HOSTS;
+
+--connection server_2
+--echo # Resume slave so it can ACK the transaction
+set debug_sync= "now SIGNAL reply_ack_to_master";
+
+--echo # Waiting for Gtid_State_Ack to reflect latest transaction for all replicas..
+let $show_statement= SHOW REPLICA HOSTS;
+let $field= Gtid_State_Ack;
+let $condition= LIKE '$binlog_gtid';
+let $wait_for_all= 1;
+source include/wait_show_condition.inc;
+
+--connection server_1
+--replace_result $SLAVE_MYPORT SLAVE_PORT $SERVER_MYPORT_3 SLAVE2_PORT $DEFAULT_MASTER_PORT DEFAULT_PORT
+SHOW REPLICA HOSTS;
+
+--connection server_2
+--source include/stop_slave.inc
+set @@GLOBAL.debug_dbug= @save_s2_debug;
+SET debug_sync='RESET';
+--source include/start_slave.inc
+
+--connection server_3
+--source include/stop_slave.inc
+set @@GLOBAL.debug_dbug= @save_s3_debug;
+SET debug_sync='RESET';
+--source include/start_slave.inc
+
+--connection server_1
+--source include/save_master_gtid.inc
+--connection server_2
+--source include/sync_with_master_gtid.inc
+--connection server_3
+--source include/sync_with_master_gtid.inc
+
+
+--echo #
+--echo # 21322.9: If a server is behind when connecting to a primary (i.e. the
+--echo # primary has newer transactions), the replica Sync_Status should
+--echo # present as semi-sync stale and the Gtid_State_Ack should not populate
+--echo # until it has reached Semi-Sync Active
+--echo #
+
+--connection server_2
+--source include/stop_slave.inc
+
+--connection server_1
+--let $nextval= `SELECT max(a)+1 from t1`
+--eval insert into t1 values ($nextval)
+--source include/save_master_gtid.inc
+--let $binlog_gtid= `SELECT @@gtid_binlog_pos`
+--echo # Pause dump_thread of server_2 (server_3 won't be affected as it has
+--echo # already successfully ACKed the new transaction)
+set @@global.debug_dbug= "+d,pause_dump_thread_after_sending_next_full_trx";
+
+--connection server_2
+--source include/start_slave.inc
+
+--connection server_1
+set debug_sync= 'now WAIT_FOR dump_thread_paused';
+
+--echo # Ensure Gtid_State_Sent is updated to represent new transaction has
+--echo # been sent to both replicas
+let $show_statement= SHOW REPLICA HOSTS;
+let $field= Gtid_State_Sent;
+let $condition= LIKE '$binlog_gtid';
+let $wait_for_all= 1;
+source include/wait_show_condition.inc;
+
+--echo # Ensure Sync_Status is Semi-sync Stale for the debug_sync held dump
+--echo # thread, as it hasn't yet got "up-to-date"
+let $show_statement= SHOW REPLICA HOSTS;
+let $field= Sync_Status;
+let $condition= LIKE 'Semi-sync Stale';
+let $wait_for_all= 0;
+source include/wait_show_condition.inc;
+
+set debug_sync= 'now SIGNAL dump_thread_continue';
+
+--echo # Ensure Sync_Status will automatically update to Semi-sync Active
+--echo # once the last stale transaction has finished sending
+let $show_statement= SHOW REPLICA HOSTS;
+let $field= Sync_Status;
+let $condition= LIKE 'Semi-sync Active';
+let $wait_for_all= 1;
+source include/wait_show_condition.inc;
+
+set @@global.debug_dbug= @save_primary_dbug;
+SET debug_sync='RESET';
+
+--source include/save_master_gtid.inc
+--connection server_2
+--source include/sync_with_master_gtid.inc
+--connection server_3
+--source include/sync_with_master_gtid.inc
+
+
+--echo #
+--echo # 21322.10a: If one replica errors (i.e. server_2 by injecting
+--echo # corrupt_gtid_event with debug_dbug), it shouldn't send its ACK,
+--echo # and server_3 does ACK; then Gtid_State_Sent should still reflect the
+--echo # new transaction for each replica, but Gtid_State_Ack should only be
+--echo # updated by the successful transaction. When the errored replica
+--echo # reconnects, it shouldn't ACK the transaction, so its on-reconnect
+--echo # Gtid_State_Ack value should be empty, but still have a Sync_Status
+--echo # of "Semi-sync Active" when it receives the latest transaction, as it
+--echo # will be ready to ACK new transactions going forward.
+--echo #
+--connection server_2
+--source include/stop_slave.inc
+SET @@GLOBAL.debug_dbug= "+d,corrupt_gtid_event";
+--source include/start_slave.inc
+
+--echo # Set-up server_3 for 10b (so we don't have to restart the slave)
+--connection server_3
+--source include/stop_slave.inc
+set @@GLOBAL.debug_dbug="+d,synchronize_semisync_slave_reply";
+--source include/start_slave.inc
+
+--connection server_1
+--echo # Waiting for master to recognize slave restarts..
+let $status_var= Rpl_semi_sync_master_clients;
+let $status_var_value= 2;
+source include/wait_for_status_var.inc;
+
+--connection server_1
+--let $nextval= `SELECT max(a)+1 from t1`
+--send_eval insert into t1 values ($nextval)
+
+--echo # Debug_sync is irrelevant to this testcase (10a) but we must do it to
+--echo # allow server_3 to ACK now (Note debug_sync is needed for 10b)
+--connection server_3
+set debug_sync= "now WAIT_FOR at_slave_reply";
+set debug_sync= "now SIGNAL reply_ack_to_master";
+
+--connection server_1
+--reap
+--let $binlog_gtid= `SELECT @@gtid_binlog_pos`
+--source include/save_master_gtid.inc
+
+--echo # Waiting for Gtid_State_Sent to reflect latest transaction for all replicas..
+let $show_statement= SHOW REPLICA HOSTS;
+let $field= Gtid_State_Sent;
+let $condition= LIKE '$binlog_gtid';
+let $wait_for_all= 1;
+source include/wait_show_condition.inc;
+
+--connection server_2
+--let $slave_io_errno= 1595
+--source include/wait_for_slave_io_error.inc
+set @@GLOBAL.debug_dbug= @save_s2_debug;
+
+--connection server_1
+--echo # Only server_3 should ACKed have the new GTID, server_2 should not due to corrupt_queue_event
+--replace_result $SLAVE_MYPORT SLAVE_PORT $SERVER_MYPORT_3 SLAVE2_PORT $DEFAULT_MASTER_PORT DEFAULT_PORT
+SHOW REPLICA HOSTS;
+
+--connection server_2
+--source include/start_slave.inc
+--source include/sync_with_master_gtid.inc
+
+--connection server_1
+--echo # With replica restarted/synced, its Gtid_State_Ack should be empty with Sync_Status semi-sync active
+--replace_result $SLAVE_MYPORT SLAVE_PORT $SERVER_MYPORT_3 SLAVE2_PORT $DEFAULT_MASTER_PORT DEFAULT_PORT
+SHOW REPLICA HOSTS;
+
+
+--echo #
+--echo # 21322.10b: Succeeding the previous 10a test, if server_3 now stalls
+--echo # (i.e. using debug_sync), then the previously errored server_2 should
+--echo # receive and ACK new transactions as a "lone" replica, and update its
+--echo # Gtid_State_* columns appropriately, whereas server_3's Gtid_State_Ack
+--echo # column should not be updated.
+--echo #
+
+--let $old_binlog_gtid= `SELECT @@gtid_binlog_pos`
+--let $nextval= `SELECT max(a)+1 from t1`
+--eval insert into t1 values ($nextval)
+--let $new_binlog_gtid= `SELECT @@gtid_binlog_pos`
+--source include/save_master_gtid.inc
+
+--connection server_2
+--source include/sync_with_master_gtid.inc
+
+--echo # Waiting for Gtid_State_Sent to reflect latest transaction for all replicas..
+let $show_statement= SHOW REPLICA HOSTS;
+let $field= Gtid_State_Sent;
+let $condition= LIKE '$new_binlog_gtid';
+let $wait_for_all= 1;
+source include/wait_show_condition.inc;
+
+--connection server_3
+set debug_sync= "now WAIT_FOR at_slave_reply";
+
+--connection server_1
+--echo # Only server_2 should have ACKed the new GTID ACKed, server_3 should not due to stall
+--replace_result $SLAVE_MYPORT SLAVE_PORT $SERVER_MYPORT_3 SLAVE2_PORT $DEFAULT_MASTER_PORT DEFAULT_PORT
+SHOW REPLICA HOSTS;
+
+--echo # Resume server_3
+--connection server_3
+set debug_sync= "now SIGNAL reply_ack_to_master";
+--source include/sync_with_master_gtid.inc
+SET debug_sync='RESET';
+
+--source include/stop_slave.inc
+set @@GLOBAL.debug_dbug= @save_s3_debug;
+SET debug_sync='RESET';
+--source include/start_slave.inc
+
+
+--echo #
+--echo # 21322.11: Configuration rpl_semi_sync_master_timeout of 0 should
+--echo # have transaction behavior match asynchronous behavior (i.e. trxs don't
+--echo # need to wait for ACKs), yet the slave should still send ACKs as a
+--echo # normal semi-sync replica, and Gtid_State_Ack should still be updated
+--echo # accordingly.
+--echo #
+--echo # The actual behavior tested in this case is as follows:
+--echo # a) Transactions won't await ACKs to complete
+--echo # b) Semi-sync remains ON when a transaction completes without an ACK
+--echo # c) Gtid_State_Ack is updated accordingly for each replica's ACK (even
+--echo # when it is behind). Here, we hold both replicas using DEBUG_SYNC to
+--echo # not send their ACKs, meanwhile, we continue creating transactions
+--echo # on the primary.
+--echo # d) If a "very lagged" replica sends an ACK for a transaction from a
+--echo # purged binlog, the Gtid_State_Ack value should be cleared, and
+--echo # issue a warning to the user with the slave's last ACKed binlog
+--echo # coordinate (i.e. filename and position).
+--echo #
+
+--connection server_1
+set global rpl_semi_sync_master_timeout=0;
+
+--connection server_2
+--source include/stop_slave.inc
+SET @@GLOBAL.DEBUG_DBUG="+d,synchronize_semisync_slave_reply";
+--source include/start_slave.inc
+--connection server_3
+--source include/stop_slave.inc
+SET @@GLOBAL.DEBUG_DBUG="+d,synchronize_semisync_slave_reply";
+--source include/start_slave.inc
+
+--connection server_1
+--echo # Waiting for master to recognize slave restarts..
+let $status_var= Rpl_semi_sync_master_clients;
+let $status_var_value= 2;
+source include/wait_for_status_var.inc;
+
+--echo #
+--echo # 21322.11.a
+connection server_1;
+--let $yes_tx_orig= query_get_value(show status like 'Rpl_semi_sync_master_yes_tx', Value, 1)
+--let $no_tx_orig= query_get_value(show status like 'Rpl_semi_sync_master_no_tx', Value, 1)
+--let $nextval= `SELECT max(a)+1 from t1`
+--eval insert into t1 values ($nextval)
+--let $trx1_gtid= `SELECT @@gtid_binlog_pos`
+
+--echo # Waiting for Gtid_State_Sent to reflect latest transaction for all replicas..
+let $show_statement= SHOW REPLICA HOSTS;
+let $field= Gtid_State_Sent;
+let $condition= LIKE '$trx1_gtid';
+let $wait_for_all= 1;
+source include/wait_show_condition.inc;
+
+--connection server_2
+SET debug_sync='now WAIT_FOR at_slave_reply';
+--connection server_3
+SET debug_sync='now WAIT_FOR at_slave_reply';
+
+--echo # Gtid_State_Ack should be empty for both replicas (as they were restarted)..
+--let $binlog_gtid= `SELECT @@gtid_binlog_pos`
+let $show_statement= SHOW REPLICA HOSTS;
+let $field= Gtid_State_Ack;
+let $condition= = '';
+let $wait_for_all= 1;
+source include/wait_show_condition.inc;
+
+--echo #
+--echo # 21322.11.b
+--connection server_1
+--echo # Ensuring semi-sync status on primary is correct..
+--let $yes_tx_post_commit= query_get_value(show status like 'Rpl_semi_sync_master_yes_tx', Value, 1)
+if ($yes_tx_orig != $yes_tx_post_commit)
+{
+ --echo # yes_tx original: $yes_tx_orig
+ --echo # yes_tx after commit: $yes_tx_post_commit
+ --die Rpl_semi_sync_master_yes_tx should not have changed when Rpl_semi_sync_master_timeout is 0
+}
+--let $no_tx_post_commit= query_get_value(show status like 'Rpl_semi_sync_master_no_tx', Value, 1)
+if ($no_tx_orig != $no_tx_post_commit)
+{
+ --echo # no_tx original: $no_tx_orig
+ --echo # no_tx after commit: $no_tx_post_commit
+ --die Rpl_semi_sync_master_no_tx should not have changed when Rpl_semi_sync_master_timeout is 0
+}
+--let $master_semisync_status= query_get_value(show status like 'Rpl_semi_sync_master_status', Value, 1)
+if (`SELECT strcmp('$master_semisync_status','ON') != 0`)
+{
+ --die Master semi-sync status was disabled after transaction
+}
+
+--echo #
+--echo # 21322.11.c
+--connection server_1
+--let $nextval= `SELECT max(a)+1 from t1`
+--eval insert into t1 values ($nextval)
+--let $trx2_gtid= `SELECT @@gtid_binlog_pos`
+--source include/save_master_gtid.inc
+
+--echo # Waiting for Gtid_State_Sent to reflect latest transaction for all replicas..
+let $show_statement= SHOW REPLICA HOSTS;
+let $field= Gtid_State_Sent;
+let $condition= LIKE '$trx2_gtid';
+let $wait_for_all= 1;
+source include/wait_show_condition.inc;
+
+--connection server_1
+--echo # server_2 and 3 should both show an empty ACK state
+--replace_result $SLAVE_MYPORT SLAVE_PORT $SERVER_MYPORT_3 SLAVE2_PORT $DEFAULT_MASTER_PORT DEFAULT_PORT $trx1_gtid TRX1_GTID $trx2_gtid TRX2_GTID
+SHOW REPLICA HOSTS;
+
+--echo # Let server_2 ACK just the first transaction
+--connection server_2
+set debug_sync= "now SIGNAL reply_ack_to_master";
+set debug_sync= "now WAIT_FOR at_slave_reply";
+
+--connection server_1
+--echo # Waiting for server_2 Gtid_State_Ack to reflect first transaction
+let $show_statement= SHOW REPLICA HOSTS;
+let $field= Gtid_State_Ack;
+let $condition= LIKE '$trx1_gtid';
+let $wait_for_all= 0;
+source include/wait_show_condition.inc;
+
+--echo # Let server_2 ACK the second transaction
+--connection server_2
+set debug_sync= "now SIGNAL reply_ack_to_master";
+--source include/sync_with_master_gtid.inc
+
+--connection server_1
+--echo # Waiting for server_2 Gtid_State_Ack to reflect second transaction
+let $show_statement= SHOW REPLICA HOSTS;
+let $field= Gtid_State_Ack;
+let $condition= LIKE '$trx2_gtid';
+let $wait_for_all= 0;
+source include/wait_show_condition.inc;
+
+--echo # Let server_3 now ACK the first transaction
+--connection server_3
+set debug_sync= "now SIGNAL reply_ack_to_master";
+set debug_sync= "now WAIT_FOR at_slave_reply";
+
+--connection server_1
+--echo # Waiting for server_3 Gtid_State_Ack to reflect first transaction
+let $show_statement= SHOW REPLICA HOSTS;
+let $field= Gtid_State_Ack;
+let $condition= LIKE '$trx1_gtid';
+let $wait_for_all= 0;
+source include/wait_show_condition.inc;
+
+--echo # Let server_3 ACK the second transaction
+--connection server_3
+set debug_sync= "now SIGNAL reply_ack_to_master";
+--source include/sync_with_master_gtid.inc
+
+--connection server_1
+--echo # Waiting for Gtid_State_Ack to reflect second transaction for both servers
+let $show_statement= SHOW REPLICA HOSTS;
+let $field= Gtid_State_Ack;
+let $condition= LIKE '$trx2_gtid';
+let $wait_for_all= 1;
+source include/wait_show_condition.inc;
+
+
+--echo #
+--echo # 21322.11.d
+--connection server_1
+--let $trx2_binlog_file= query_get_value(SHOW BINARY LOGS, Log_name, 1)
+--let $trx2_binlog_pos= query_get_value(SHOW BINARY LOGS, File_size, 1)
+FLUSH LOGS;
+--let $nextval= `SELECT max(a)+1 from t1`
+--eval insert into t1 values ($nextval)
+--let $trx3_gtid= `SELECT @@gtid_binlog_pos`
+--source include/save_master_gtid.inc
+
+--echo # Waiting for Gtid_State_Sent to reflect latest transaction for all replicas..
+let $show_statement= SHOW REPLICA HOSTS;
+let $field= Gtid_State_Sent;
+let $condition= LIKE '$trx3_gtid';
+let $wait_for_all= 1;
+source include/wait_show_condition.inc;
+
+--connection server_2
+set debug_sync= "now WAIT_FOR at_slave_reply";
+--connection server_3
+set debug_sync= "now WAIT_FOR at_slave_reply";
+
+--connection server_1
+--echo # server_2 and 3 should both show ACKed TRX2 (with TRX3 Sent)
+--replace_result $SLAVE_MYPORT SLAVE_PORT $SERVER_MYPORT_3 SLAVE2_PORT $DEFAULT_MASTER_PORT DEFAULT_PORT $trx2_gtid TRX2_GTID $trx3_gtid TRX3_GTID
+SHOW REPLICA HOSTS;
+
+--let $purge_to_binlog= query_get_value(SHOW MASTER STATUS, File, 1)
+--let $purge_binlogs_to=$purge_to_binlog
+--source include/wait_for_purge.inc
+
+--echo # Master should warn that the binary log which contains the last ACKed
+--echo # binlog coordinates has been purged, and clear Gtid_State_Ack
+--enable_warnings
+--replace_result $SLAVE_MYPORT SLAVE_PORT $SERVER_MYPORT_3 SLAVE2_PORT $DEFAULT_MASTER_PORT DEFAULT_PORT $trx2_gtid TRX2_GTID $trx3_gtid TRX3_GTID $trx2_binlog_file TRX2_BINLOG_FILE $trx2_binlog_pos TRX2_BINLOG_POS
+SHOW REPLICA HOSTS;
+
+--echo # Let servers ACK new transaction
+--connection server_2
+set debug_sync= "now SIGNAL reply_ack_to_master";
+--connection server_3
+set debug_sync= "now SIGNAL reply_ack_to_master";
+
+--connection server_1
+--echo # Waiting for Gtid_State_Ack to reflect latest transaction for all replicas..
+let $show_statement= SHOW REPLICA HOSTS;
+let $field= Gtid_State_Ack;
+let $condition= LIKE '$trx3_gtid';
+let $wait_for_all= 1;
+source include/wait_show_condition.inc;
+
+--echo # Gtid_State_Ack should now show the latest transaction GTID
+--replace_result $SLAVE_MYPORT SLAVE_PORT $SERVER_MYPORT_3 SLAVE2_PORT $DEFAULT_MASTER_PORT DEFAULT_PORT $trx2_gtid TRX2_GTID $trx3_gtid TRX3_GTID
+SHOW REPLICA HOSTS;
+--disable_warnings
+
+--connection server_2
+--source include/sync_with_master_gtid.inc
+--connection server_3
+--source include/sync_with_master_gtid.inc
+
+
+--echo #
+--echo # MDEV-21322 Cleanup
+connection server_1;
+set @@global.debug_dbug= @save_primary_dbug;
+set @@global.rpl_semi_sync_master_timeout= @save_semisync_timeout;
+set @@global.rpl_semi_sync_master_enabled= @save_semisync_master_enabled;
+
+drop table t1;
+--source include/save_master_gtid.inc
+--save_master_pos
+
+connection server_2;
+--source include/sync_with_master_gtid.inc
+--source include/stop_slave.inc
+set @@global.rpl_semi_sync_slave_enabled= @save_semisync_server_2_enabled;
+--eval SET @@GLOBAL.debug_dbug= "$save_server_2_dbug"
+SET debug_sync='RESET';
+--source include/start_slave.inc
+
+connection server_3;
+--source include/sync_with_master_gtid.inc
+--source include/stop_slave.inc
+set @@global.rpl_semi_sync_slave_enabled= @save_semisync_server_3_enabled;
+--eval SET @@GLOBAL.debug_dbug= "$save_server_3_dbug"
+SET debug_sync='RESET';
+--source include/start_slave.inc
+
+--echo #
+--echo # End of MDEV-21322 tests
+--echo #
+
+# End of rpl_show_slave_hosts.test
--source include/rpl_end.inc
diff --git a/sql/repl_failsafe.cc b/sql/repl_failsafe.cc
index d0285b54928..67a12cee80d 100644
--- a/sql/repl_failsafe.cc
+++ b/sql/repl_failsafe.cc
@@ -37,18 +37,7 @@
#include "rpl_filter.h"
#include "log_event.h"
#include <mysql.h>
-
-
-struct Slave_info
-{
- uint32 server_id;
- uint32 master_id;
- char host[HOSTNAME_LENGTH*SYSTEM_CHARSET_MBMAXLEN+1];
- char user[USERNAME_LENGTH+1];
- char password[MAX_PASSWORD_LENGTH*SYSTEM_CHARSET_MBMAXLEN+1];
- uint16 port;
-};
-
+#include "semisync_master.h"
Atomic_counter<uint32_t> binlog_dump_thread_count;
ulong rpl_status=RPL_NULL;
@@ -125,8 +114,11 @@ int THD::register_slave(uchar *packet, size_t packet_length)
if (check_access(this, PRIV_COM_REGISTER_SLAVE, any_db.str, NULL,NULL,0,0))
return 1;
if (!(si= (Slave_info*)my_malloc(key_memory_SLAVE_INFO, sizeof(Slave_info),
- MYF(MY_WME))))
+ MYF(MY_WME|MY_ZEROFILL))))
return 1;
+ memset(si->gtid_state_sent.log_file, '\0', FN_REFLEN);
+ memset(si->gtid_state_ack.log_file, '\0', FN_REFLEN);
+ si->sync_status= Slave_info::SYNC_STATE_INITIALIZING;
variables.server_id= si->server_id= uint4korr(p);
p+= 4;
@@ -179,7 +171,10 @@ static my_bool show_slave_hosts_callback(THD *thd, Protocol *protocol)
{
my_bool res= FALSE;
mysql_mutex_lock(&thd->LOCK_thd_data);
- if (auto si= thd->slave_info)
+ String gtid_sent, gtid_ack;
+ const char *sync_str;
+ const char *err_msg= NULL;
+ if (const Slave_info *si= thd->slave_info)
{
protocol->prepare_for_resend();
protocol->store(si->server_id);
@@ -191,6 +186,50 @@ static my_bool show_slave_hosts_callback(THD *thd, Protocol *protocol)
}
protocol->store((uint32) si->port);
protocol->store(si->master_id);
+
+ if (gtid_state_from_binlog_pos(si->gtid_state_sent.log_file,
+ (uint32) si->gtid_state_sent.log_pos,
+ >id_sent, &err_msg))
+ {
+ gtid_sent.length(0);
+ DBUG_ASSERT(err_msg);
+ if (global_system_variables.log_warnings >= 2)
+ push_warning_printf(
+ current_thd, Sql_condition::WARN_LEVEL_WARN,
+ ER_MASTER_CANNOT_RECONSTRUCT_GTID_STATE_FOR_BINLOG_POS,
+ ER_THD(current_thd,
+ ER_MASTER_CANNOT_RECONSTRUCT_GTID_STATE_FOR_BINLOG_POS),
+ si->gtid_state_sent.log_pos, si->gtid_state_sent.log_file,
+ err_msg);
+ }
+ protocol->store(>id_sent);
+
+ if (rpl_semi_sync_master_enabled && thd->semi_sync_slave)
+ {
+ if (gtid_state_from_binlog_pos(si->gtid_state_ack.log_file,
+ (uint32) si->gtid_state_ack.log_pos,
+ >id_ack, &err_msg))
+ {
+ gtid_ack.length(0);
+ DBUG_ASSERT(err_msg);
+
+ if (global_system_variables.log_warnings >= 2)
+ {
+ push_warning_printf(
+ current_thd, Sql_condition::WARN_LEVEL_WARN,
+ ER_MASTER_CANNOT_RECONSTRUCT_GTID_STATE_FOR_BINLOG_POS,
+ ER_THD(current_thd,
+ ER_MASTER_CANNOT_RECONSTRUCT_GTID_STATE_FOR_BINLOG_POS),
+ si->gtid_state_ack.log_pos, si->gtid_state_ack.log_file,
+ err_msg);
+ }
+ }
+ }
+ protocol->store(>id_ack);
+
+ sync_str= si->get_sync_status_str();
+ protocol->store(sync_str, safe_strlen(sync_str), &my_charset_bin);
+
res= protocol->write();
}
mysql_mutex_unlock(&thd->LOCK_thd_data);
@@ -235,6 +274,20 @@ bool show_slave_hosts(THD* thd)
Item_return_int(thd, "Master_id", 10, MYSQL_TYPE_LONG),
thd->mem_root);
+ /* Length matches GTID_IO_Pos of SHOW SLAVE STATUS on slave */
+ field_list.push_back(new (mem_root)
+ Item_empty_string(thd, "Gtid_State_Sent", 30),
+ thd->mem_root);
+
+ field_list.push_back(new (mem_root)
+ Item_empty_string(thd, "Gtid_State_Ack", 30),
+ thd->mem_root);
+
+ /* For the length, use the size of the longest possible value */
+ field_list.push_back(new (mem_root) Item_empty_string(
+ thd, "Sync_Status", sizeof("Semi-sync Active")),
+ thd->mem_root);
+
if (protocol->send_result_set_metadata(&field_list,
Protocol::SEND_NUM_ROWS | Protocol::SEND_EOF))
DBUG_RETURN(TRUE);
diff --git a/sql/semisync_master.cc b/sql/semisync_master.cc
index fdf2cf21cf1..8b07c25040e 100644
--- a/sql/semisync_master.cc
+++ b/sql/semisync_master.cc
@@ -53,13 +53,6 @@ ulonglong rpl_semi_sync_master_trx_wait_time = 0;
Repl_semi_sync_master repl_semisync_master;
Ack_receiver ack_receiver;
-/*
- structure to save transaction log filename and position
-*/
-typedef struct Trans_binlog_info {
- my_off_t log_pos;
- char log_file[FN_REFLEN];
-} Trans_binlog_info;
static int get_wait_time(const struct timespec& start_ts);
@@ -591,7 +584,7 @@ void Repl_semi_sync_master::remove_slave()
@retval -1 Slave is going down (ok)
*/
-int Repl_semi_sync_master::report_reply_packet(uint32 server_id,
+int Repl_semi_sync_master::report_reply_packet(Slave_info *slave_info,
const uchar *packet,
ulong packet_len)
{
@@ -635,12 +628,12 @@ int Repl_semi_sync_master::report_reply_packet(uint32 server_id,
DBUG_ASSERT(dirname_length(log_file_name) == 0);
- DBUG_PRINT("semisync", ("%s: Got reply(%s, %lu) from server %u",
- "Repl_semi_sync_master::report_reply_packet",
- log_file_name, (ulong)log_file_pos, server_id));
-
+ DBUG_PRINT("semisync",
+ ("%s: Got reply(%s, %lu) from server %u",
+ "Repl_semi_sync_master::report_reply_packet", log_file_name,
+ (ulong) log_file_pos, slave_info->server_id));
rpl_semi_sync_master_get_ack++;
- report_reply_binlog(server_id, log_file_name, log_file_pos);
+ report_reply_binlog(slave_info, log_file_name, log_file_pos);
DBUG_RETURN(0);
l_end:
@@ -649,13 +642,13 @@ int Repl_semi_sync_master::report_reply_packet(uint32 server_id,
octet2hex(buf, (const unsigned char*) packet,
MY_MIN(sizeof(buf)-1, (size_t) packet_len));
sql_print_information("First bytes of the packet from semisync slave "
- "server-id %d: %s", server_id, buf);
+ "server-id %d: %s", slave_info->server_id, buf);
}
DBUG_RETURN(result);
}
-int Repl_semi_sync_master::report_reply_binlog(uint32 server_id,
+int Repl_semi_sync_master::report_reply_binlog(Slave_info *slave_info,
const char *log_file_name,
my_off_t log_file_pos)
{
@@ -675,7 +668,7 @@ int Repl_semi_sync_master::report_reply_binlog(uint32 server_id,
if (!is_on())
/* We check to see whether we can switch semi-sync ON. */
- try_switch_on(server_id, log_file_name, log_file_pos);
+ try_switch_on(slave_info->server_id, log_file_name, log_file_pos);
/* The position should increase monotonically, if there is only one
* thread sending the binlog to the slave.
@@ -719,6 +712,19 @@ int Repl_semi_sync_master::report_reply_binlog(uint32 server_id,
DBUG_PRINT("semisync", ("%s: Got reply at (%s, %lu)",
"Repl_semi_sync_master::report_reply_binlog",
log_file_name, (ulong)log_file_pos));
+ goto update_gtid_state_ack;
+ }
+ else if (rpl_semi_sync_master_clients > 1 &&
+ Active_tranx::compare(slave_info->gtid_state_ack.log_file,
+ slave_info->gtid_state_ack.log_pos,
+ m_reply_file_name, m_reply_file_pos))
+ {
+update_gtid_state_ack:
+ /*
+ Each slave should still maintain its Gtid_state_ack
+ */
+ strncpy(slave_info->gtid_state_ack.log_file, log_file_name, strlen(log_file_name));
+ slave_info->gtid_state_ack.log_pos= log_file_pos;
}
@@ -828,7 +834,7 @@ int Repl_semi_sync_master::dump_start(THD* thd,
}
add_slave();
- report_reply_binlog(thd->variables.server_id,
+ report_reply_binlog(thd->slave_info,
log_file + dirname_length(log_file), log_pos);
sql_print_information("Start semi-sync binlog_dump to slave "
"(server_id: %ld), pos(%s, %lu)",
@@ -858,6 +864,16 @@ int Repl_semi_sync_master::commit_trx(const char *trx_wait_binlog_name,
bool success= 0;
DBUG_ENTER("Repl_semi_sync_master::commit_trx");
+ /*
+ If the semi-sync timeout is set to 0, we effectively are configured for
+ asynchronous replication; except we still want to request/receive ACKs from
+ slaves so we can monitor replication status via SHOW SLAVE HOSTS columns
+ Gtid_State_Sent and Gtid_State_Ack. Thereby, we should quit now before
+ updating rpl_semi_sync_master_(no/yes)_transactions.
+ */
+ if (!m_wait_timeout)
+ DBUG_RETURN(0);
+
if (!rpl_semi_sync_master_clients && !rpl_semi_sync_master_wait_no_slave)
{
rpl_semi_sync_master_no_transactions++;
@@ -1235,6 +1251,14 @@ int Repl_semi_sync_master::update_sync_header(THD* thd, unsigned char *packet,
*need_sync= sync;
l_end:
+ if (is_on())
+ {
+ thd->slave_info->sync_status=
+ sync ? thd->slave_info->sync_status=
+ Slave_info::SYNC_STATE_SEMI_SYNC_ACTIVE
+ : thd->slave_info->sync_status=
+ Slave_info::SYNC_STATE_SEMI_SYNC_STALE;
+ }
unlock();
/*
diff --git a/sql/semisync_master.h b/sql/semisync_master.h
index 3978d21a61d..cfb5e3d150b 100644
--- a/sql/semisync_master.h
+++ b/sql/semisync_master.h
@@ -28,6 +28,91 @@ extern PSI_mutex_key key_LOCK_binlog;
extern PSI_cond_key key_COND_binlog_send;
#endif
+
+/*
+ structure to save transaction log filename and position
+*/
+typedef struct Trans_binlog_info {
+ my_off_t log_pos;
+ char log_file[FN_REFLEN];
+} Trans_binlog_info;
+
+
+struct Slave_info
+{
+public:
+ enum synchronization_status {
+ /*
+ Binlog dump thread is initializing, we don't yet know the synchronization
+ status
+ */
+ SYNC_STATE_INITIALIZING,
+
+ /*
+ Slave is asynchronous, so Gtid_State_Ack will not be updated
+ */
+ SYNC_STATE_ASYNCHRONOUS,
+
+ /*
+ Slave is configured for semi-sync, but connected with an old state, and
+ is catching up now
+ */
+ SYNC_STATE_SEMI_SYNC_STALE,
+
+ /*
+ Slave is configured for semi-sync, and is readily ACKing new transactions
+ */
+ SYNC_STATE_SEMI_SYNC_ACTIVE
+ };
+
+ uint32 server_id;
+ uint32 master_id;
+ char host[HOSTNAME_LENGTH*SYSTEM_CHARSET_MBMAXLEN+1];
+ char user[USERNAME_LENGTH+1];
+ char password[MAX_PASSWORD_LENGTH*SYSTEM_CHARSET_MBMAXLEN+1];
+ uint16 port;
+
+ /*
+ Binlog file:pos of the last transaction sent to this replica. Used to infer
+ Gtid_State_Sent in SHOW REPLICA HOSTS. Used for both asynchronous and
+ semi-sync connections.
+ */
+ Trans_binlog_info gtid_state_sent;
+
+ /*
+ If replica is configured for semi-sync, the binlog file:pos of the last
+ transaction ACKed by this replica. Used to infer Gtid_State_Ack in
+ SHOW REPLICA HOSTS.
+ */
+ Trans_binlog_info gtid_state_ack;
+
+ /*
+ Sync_Status of SHOW REPLICA HOSTS.
+ */
+ synchronization_status sync_status;
+
+ const char *get_sync_status_str() const
+ {
+ const char *ret;
+ switch (sync_status)
+ {
+ case SYNC_STATE_INITIALIZING:
+ ret= "Initializing";
+ break;
+ case SYNC_STATE_ASYNCHRONOUS:
+ ret= "Asynchronous";
+ break;
+ case SYNC_STATE_SEMI_SYNC_STALE:
+ ret= "Semi-sync Stale";
+ break;
+ default:
+ ret= "Semi-sync Active";
+ }
+ return ret;
+ }
+};
+
+
struct Tranx_node {
char log_name[FN_REFLEN];
my_off_t log_pos;
@@ -561,14 +646,14 @@ class Repl_semi_sync_master
void remove_slave();
/* It parses a reply packet and call report_reply_binlog to handle it. */
- int report_reply_packet(uint32 server_id, const uchar *packet,
- ulong packet_len);
+ int report_reply_packet(Slave_info *slave_info, const uchar *packet,
+ ulong packet_len);
/* In semi-sync replication, reports up to which binlog position we have
* received replies from the slave indicating that it already get the events.
*
* Input:
- * server_id - (IN) master server id number
+ * slave_info - (IN) info of the slave which sent the ACK
* log_file_name - (IN) binlog file name
* end_offset - (IN) the offset in the binlog file up to which we have
* the replies from the slave
@@ -576,7 +661,7 @@ class Repl_semi_sync_master
* Return:
* 0: success; non-zero: error
*/
- int report_reply_binlog(uint32 server_id,
+ int report_reply_binlog(Slave_info *slave_info,
const char* log_file_name,
my_off_t end_offset);
diff --git a/sql/semisync_master_ack_receiver.cc b/sql/semisync_master_ack_receiver.cc
index 29fa5fd5328..ba2ea0c9762 100644
--- a/sql/semisync_master_ack_receiver.cc
+++ b/sql/semisync_master_ack_receiver.cc
@@ -16,6 +16,7 @@
#include <my_global.h>
#include "semisync_master.h"
#include "semisync_master_ack_receiver.h"
+#include "debug_sync.h"
#ifdef HAVE_PSI_MUTEX_INTERFACE
extern PSI_mutex_key key_LOCK_ack_receiver;
@@ -353,7 +354,20 @@ void Ack_receiver::run()
if (likely(len != packet_error))
{
int res;
- res= repl_semisync_master.report_reply_packet(slave->server_id(),
+#ifdef ENABLED_DEBUG_SYNC
+ /*
+ A (+d,pause_ack_thread_on_next_ack)-test is supposed to
+ be run to check `Gtid_state_ack` in show replica hosts
+ for cases where there are multiple active replicas.
+ */
+ DBUG_EXECUTE_IF("pause_ack_thread_on_next_ack",
+ {
+ const char act[]= "now SIGNAL pause_ack_reply_to_binlog WAIT_FOR resume_ack_thread";
+ DBUG_ASSERT(!debug_sync_set_action(thd, STRING_WITH_LEN(act)));
+ DBUG_SET("-d,pause_ack_thread_on_next_ack");
+ };);
+#endif
+ res= repl_semisync_master.report_reply_packet(slave->thd->slave_info,
net.read_pos, len);
if (unlikely(res < 0))
{
diff --git a/sql/share/errmsg-utf8.txt b/sql/share/errmsg-utf8.txt
index 22d0f4d3392..a428c1c63f8 100644
--- a/sql/share/errmsg-utf8.txt
+++ b/sql/share/errmsg-utf8.txt
@@ -12280,3 +12280,5 @@ ER_SEQUENCE_TABLE_CANNOT_HAVE_ANY_CONSTRAINTS
eng "Sequence tables cannot have any constraints"
ER_SEQUENCE_TABLE_ORDER_BY
eng "ORDER BY"
+ER_MASTER_CANNOT_RECONSTRUCT_GTID_STATE_FOR_BINLOG_POS
+ eng "Error constructing GTID state for binlog position %u in file '%s': %s"
\ No newline at end of file
diff --git a/sql/slave.cc b/sql/slave.cc
index 54e70bc7385..8b6d09d28ac 100644
--- a/sql/slave.cc
+++ b/sql/slave.cc
@@ -5121,6 +5121,13 @@ Stopping slave I/O thread due to out-of-memory error from master");
{
DBUG_EXECUTE_IF("simulate_delay_semisync_slave_reply",
my_sleep(800000););
+#ifdef ENABLED_DEBUG_SYNC
+ DBUG_EXECUTE_IF("synchronize_semisync_slave_reply",
+ {
+ const char act[]= "now SIGNAL at_slave_reply WAIT_FOR reply_ack_to_master";
+ DBUG_ASSERT(!debug_sync_set_action(thd, STRING_WITH_LEN(act)));
+ };);
+#endif
if (repl_semisync_slave.slave_reply(mi))
{
/*
@@ -6120,19 +6127,33 @@ static int queue_event(Master_info* mi, const uchar *buf, ulong event_len)
// will have to refine the clause.
DBUG_ASSERT(mi->rli.relay_log.relay_log_checksum_alg !=
BINLOG_CHECKSUM_ALG_UNDEF);
-
- // Emulate the network corruption
- DBUG_EXECUTE_IF("corrupt_queue_event",
- if (buf[EVENT_TYPE_OFFSET] != FORMAT_DESCRIPTION_EVENT)
- {
- uchar *debug_event_buf_c= const_cast<uchar*>(buf);
- int debug_cor_pos = rand() % (event_len - BINLOG_CHECKSUM_LEN);
- debug_event_buf_c[debug_cor_pos] =~ debug_event_buf_c[debug_cor_pos];
- DBUG_PRINT("info", ("Corrupt the event at queue_event: byte on position %d", debug_cor_pos));
- DBUG_SET("-d,corrupt_queue_event");
- }
- );
-
+
+#ifndef DBUG_OFF
+ {
+ const char *dbug_unset;
+ // Emulate the network corruption
+ DBUG_EXECUTE_IF(
+ "corrupt_gtid_event",
+ if (buf[EVENT_TYPE_OFFSET] == GTID_EVENT) {
+ dbug_unset= "-d,corrupt_gtid_event";
+ goto corrupt_event;
+ });
+ DBUG_EXECUTE_IF(
+ "corrupt_queue_event",
+ if (buf[EVENT_TYPE_OFFSET] != FORMAT_DESCRIPTION_EVENT) {
+ dbug_unset= "-d,corrupt_queue_event";
+ corrupt_event:
+ uchar *debug_event_buf_c= const_cast<uchar *>(buf);
+ int debug_cor_pos= rand() % (event_len - BINLOG_CHECKSUM_LEN);
+ debug_event_buf_c[debug_cor_pos]= ~debug_event_buf_c[debug_cor_pos];
+ DBUG_PRINT("info",
+ ("Corrupt the event at queue_event: byte on position %d",
+ debug_cor_pos));
+ DBUG_SET(dbug_unset);
+ });
+ }
+#endif
+
if (event_checksum_test((uchar*) buf, event_len, checksum_alg))
{
error= ER_NETWORK_READ_EVENT_CHECKSUM_FAILURE;
diff --git a/sql/sql_repl.cc b/sql/sql_repl.cc
index 8ca252c37c6..18a52dc07ad 100644
--- a/sql/sql_repl.cc
+++ b/sql/sql_repl.cc
@@ -1830,18 +1830,26 @@ gtid_state_from_pos(const char *name, uint32 offset,
return errormsg;
}
-
-int
-gtid_state_from_binlog_pos(const char *in_name, uint32 pos, String *out_str)
+int gtid_state_from_binlog_pos(const char *in_name, uint32 pos,
+ String *out_str, const char **out_err)
{
slave_connection_state gtid_state;
const char *lookup_name;
char name_buf[FN_REFLEN];
LOG_INFO linfo;
+ const char *dummy_err;
+ const char **err_save;
+ int find_err= 0;
+
+ if (out_err)
+ err_save= out_err;
+ else
+ err_save= &dummy_err;
if (!mysql_bin_log.is_open())
{
my_error(ER_NO_BINARY_LOGGING, MYF(0));
+ *err_save= "Binary logging is disabled.";
return 1;
}
@@ -1853,15 +1861,27 @@ gtid_state_from_binlog_pos(const char *in_name, uint32 pos, String *out_str)
else
lookup_name= NULL;
linfo.index_file_offset= 0;
- if (mysql_bin_log.find_log_pos(&linfo, lookup_name, 1))
+ if ((find_err= mysql_bin_log.find_log_pos(&linfo, lookup_name, 1)))
+ {
+ if (find_err == LOG_INFO_EOF)
+ *err_save= "Could not find binary log file. Probably the slave state is "
+ "too old and required binlog files have been purged.";
+ else
+ *err_save= "Error reading index file.";
return 1;
+ }
if (pos < 4)
pos= 4;
- if (gtid_state_from_pos(linfo.log_file_name, pos, >id_state) ||
+ if ((*err_save=
+ gtid_state_from_pos(linfo.log_file_name, pos, >id_state)) ||
gtid_state.to_string(out_str))
+ {
+ if (!*err_save)
+ *err_save= "Failed converting GTID state to string representation.";
return 1;
+ }
return 0;
}
@@ -2272,6 +2292,30 @@ send_event_to_slave(binlog_send_info *info, Log_event_type event_type,
return "Failed to run hook 'after_send_event'";
}
+ if (info->thd->slave_info)
+ {
+ strncpy(info->thd->slave_info->gtid_state_sent.log_file,
+ info->log_file_name + info->dirlen,
+ strlen(info->log_file_name) - info->dirlen);
+ info->thd->slave_info->gtid_state_sent.log_pos= pos;
+ }
+
+#ifdef ENABLED_DEBUG_SYNC
+ DBUG_EXECUTE_IF("pause_dump_thread_after_sending_next_full_trx", {
+ if (event_type == XID_EVENT ||
+ (event_type == QUERY_EVENT &&
+ Query_log_event::peek_is_commit_rollback(
+ (uchar *) packet->ptr() + ev_offset, len - ev_offset,
+ current_checksum_alg)))
+ {
+ DBUG_ASSERT(!debug_sync_set_action(
+ info->thd, STRING_WITH_LEN("now SIGNAL dump_thread_paused "
+ "WAIT_FOR dump_thread_continue")));
+ DBUG_SET("-d,pause_dump_thread_after_sending_next_full_trx");
+ }
+ });
+#endif
+
return NULL; /* Success */
}
@@ -2753,6 +2797,10 @@ static int wait_new_events(binlog_send_info *info, /* in */
break;
}
+ if (info->thd->semi_sync_slave)
+ info->thd->slave_info->sync_status=
+ Slave_info::SYNC_STATE_SEMI_SYNC_ACTIVE;
+
if (info->heartbeat_period)
{
struct timespec ts;
@@ -3087,6 +3135,14 @@ void mysql_binlog_send(THD* thd, char* log_ident, my_off_t pos,
/* Check if the dump thread is created by a slave with semisync enabled. */
thd->semi_sync_slave = is_semi_sync_slave();
+ /*
+ If the slave is not set up for a semi-sync connection, we can tag it
+ immediately as asynchronous. Otherwise, we need to wait and see if the
+ replica is up-to-date or not to mark semi-sync active vs stale.
+ */
+ if (thd->slave_info && !thd->semi_sync_slave)
+ thd->slave_info->sync_status= Slave_info::SYNC_STATE_ASYNCHRONOUS;
+
DBUG_ASSERT(pos == linfo.pos);
if (repl_semisync_master.dump_start(thd, linfo.log_file_name, linfo.pos))
diff --git a/sql/sql_repl.h b/sql/sql_repl.h
index 51b6a599d5f..5e3ccc9df45 100644
--- a/sql/sql_repl.h
+++ b/sql/sql_repl.h
@@ -66,7 +66,8 @@ void rpl_init_gtid_slave_state();
void rpl_deinit_gtid_slave_state();
void rpl_init_gtid_waiting();
void rpl_deinit_gtid_waiting();
-int gtid_state_from_binlog_pos(const char *name, uint32 pos, String *out_str);
+int gtid_state_from_binlog_pos(const char *name, uint32 pos, String *out_str,
+ const char **out_err= NULL);
int rpl_append_gtid_state(String *dest, bool use_binlog);
int rpl_load_gtid_state(slave_connection_state *state, bool use_binlog);
bool rpl_gtid_pos_check(THD *thd, char *str, size_t len);
--
2.30.2
1
0

[PATCH 1/2] MDEV-33798: ROW base optimistic deadlock with concurrent writes on same table
by Kristian Nielsen 02 May '24
by Kristian Nielsen 02 May '24
02 May '24
One case is conflicting transactions T1 and T2 with different domain id, in
optimistic parallel replication in non-GTID mode. Then T2 will
wait_for_prior_commit on T1; and if T1 got a row lock wait on T2 it would
hang, as different domains caused the deadlock kill to be skipped in
thd_rpl_deadlock_check().
More generally, if we have transactions T1 and T2 in one domain/master
connection, and independently transactions U1 and U2 in another, then we can
still deadlock like this:
T1 row low wait on U2
U2 wait_for_prior_commit on U1
U1 row lock wait on T2
T2 wait_for_prior_commit on T1
This commit enforces the deadlock kill in these cases. If the waited-for
transaction is speculatively applied, then it will be deadlock killed in
case of a conflict, even if the two transactions are in different domains
or master connections.
Signed-off-by: Kristian Nielsen <knielsen(a)knielsen-hq.org>
---
mysql-test/suite/rpl/r/rpl_mdev33798.result | 143 +++++++++++++++
mysql-test/suite/rpl/t/rpl_mdev33798.cnf | 17 ++
mysql-test/suite/rpl/t/rpl_mdev33798.test | 182 ++++++++++++++++++++
sql/sql_class.cc | 42 ++++-
4 files changed, 376 insertions(+), 8 deletions(-)
create mode 100644 mysql-test/suite/rpl/r/rpl_mdev33798.result
create mode 100644 mysql-test/suite/rpl/t/rpl_mdev33798.cnf
create mode 100644 mysql-test/suite/rpl/t/rpl_mdev33798.test
diff --git a/mysql-test/suite/rpl/r/rpl_mdev33798.result b/mysql-test/suite/rpl/r/rpl_mdev33798.result
new file mode 100644
index 00000000000..8796e948546
--- /dev/null
+++ b/mysql-test/suite/rpl/r/rpl_mdev33798.result
@@ -0,0 +1,143 @@
+include/rpl_init.inc [topology=1->2,1->3]
+connect server_2b,127.0.0.1,root,,,$SERVER_MYPORT_2;
+connection server_2;
+SET @old_parallel_threads=@@GLOBAL.slave_parallel_threads;
+SET @old_parallel_mode= @@GLOBAL.slave_parallel_mode;
+SET @old_timeout= @@GLOBAL.lock_wait_timeout;
+SET @old_innodb_timeout= @@GLOBAL.innodb_lock_wait_timeout;
+include/stop_slave.inc
+SET GLOBAL slave_parallel_threads=5;
+set global slave_parallel_mode= aggressive;
+SET GLOBAL lock_wait_timeout= 86400;
+SET GLOBAL innodb_lock_wait_timeout= 86400;
+SET STATEMENT sql_log_bin=0 FOR ALTER TABLE mysql.gtid_slave_pos ENGINE=InnoDB;
+include/start_slave.inc
+connection server_1;
+CREATE TABLE t1 (a INT PRIMARY KEY, b INT) ENGINE=InnoDB;
+INSERT INTO t1 VALUES (1, 0), (2, 0), (3, 0), (4, 0), (5, 0), (6, 0), (7, 0), (8, 0);
+connection server_2;
+include/stop_slave.inc
+connection server_2b;
+BEGIN;
+SELECT * FROM t1 WHERE a=1 FOR UPDATE;
+a b
+1 0
+SELECT * FROM t1 WHERE a=5 FOR UPDATE;
+a b
+5 0
+connection server_1;
+SET SESSION gtid_domain_id= 1;
+BEGIN;
+UPDATE t1 SET b=1 WHERE a=1;
+UPDATE t1 SET b=1 WHERE a=7;
+COMMIT;
+UPDATE t1 SET b=2 WHERE a=3;
+SET SESSION gtid_domain_id=2;
+BEGIN;
+UPDATE t1 SET b=3 WHERE a=5;
+UPDATE t1 SET b=3 WHERE a=3;
+COMMIT;
+UPDATE t1 SET b=4 WHERE a=7;
+SET SESSION gtid_domain_id= 0;
+include/save_master_gtid.inc
+connection server_2;
+include/start_slave.inc
+connection server_2b;
+ROLLBACK;
+connection server_2;
+include/sync_with_master_gtid.inc
+SELECT a, (
+(a=1 AND b=1) OR
+(a=3 AND (b=2 OR b=3)) OR
+(a=5 AND b=3) OR
+(a=7 AND (b=1 OR b=4)) OR
+((a MOD 2)=0 AND b=0)) AS `ok`
+ FROM t1
+ORDER BY a;
+a ok
+1 1
+2 1
+3 1
+4 1
+5 1
+6 1
+7 1
+8 1
+connection server_3;
+include/sync_with_master_gtid.inc
+include/stop_slave.inc
+connection server_2;
+include/stop_slave.inc
+CHANGE MASTER 'm2' to master_port=MYPORT_3 , master_host='127.0.0.1', master_user='root', master_use_gtid=slave_pos;
+connection server_1;
+SET SESSION gtid_domain_id= 1;
+BEGIN;
+UPDATE t1 SET b=11 WHERE a=1;
+UPDATE t1 SET b=11 WHERE a=7;
+COMMIT;
+UPDATE t1 SET b=12 WHERE a=3;
+SET SESSION gtid_domain_id= 1;
+connection server_3;
+SET SESSION gtid_domain_id=3;
+BEGIN;
+UPDATE t1 SET b=13 WHERE a=5;
+UPDATE t1 SET b=13 WHERE a=3;
+COMMIT;
+UPDATE t1 SET b=14 WHERE a=7;
+include/save_master_gtid.inc
+connection server_2b;
+BEGIN;
+SELECT * FROM t1 WHERE a=1 FOR UPDATE;
+a b
+1 1
+SELECT * FROM t1 WHERE a=5 FOR UPDATE;
+a b
+5 3
+START ALL SLAVES;
+Warnings:
+Note 1937 SLAVE 'm2' started
+Note 1937 SLAVE '' started
+connection server_2b;
+ROLLBACK;
+connection server_1;
+include/save_master_gtid.inc
+connection server_2;
+include/sync_with_master_gtid.inc
+connection server_3;
+include/save_master_gtid.inc
+connection server_2;
+include/sync_with_master_gtid.inc
+SELECT a, (
+(a=1 AND b=11) OR
+(a=3 AND (b=12 OR b=13)) OR
+(a=5 AND b=13) OR
+(a=7 AND (b=11 OR b=14)) OR
+((a MOD 2)=0 AND b=0)) AS `ok`
+ FROM t1
+ORDER BY a;
+a ok
+1 1
+2 1
+3 1
+4 1
+5 1
+6 1
+7 1
+8 1
+SET default_master_connection = 'm2';
+include/stop_slave.inc
+RESET SLAVE 'm2' ALL;
+SET default_master_connection = '';
+connection server_3;
+include/start_slave.inc
+disconnect server_2b;
+connection server_1;
+DROP TABLE t1;
+connection server_2;
+include/stop_slave.inc
+SET GLOBAL slave_parallel_threads=@old_parallel_threads;
+set global slave_parallel_mode= @old_parallel_mode;
+SET GLOBAL lock_wait_timeout= @old_timeout;
+SET GLOBAL innodb_lock_wait_timeout= @old_innodb_timeout;
+include/start_slave.inc
+include/rpl_end.inc
diff --git a/mysql-test/suite/rpl/t/rpl_mdev33798.cnf b/mysql-test/suite/rpl/t/rpl_mdev33798.cnf
new file mode 100644
index 00000000000..8e5125ea6ca
--- /dev/null
+++ b/mysql-test/suite/rpl/t/rpl_mdev33798.cnf
@@ -0,0 +1,17 @@
+!include suite/rpl/my.cnf
+
+[mysqld.1]
+log-slave-updates
+loose-innodb
+
+[mysqld.2]
+log-slave-updates
+loose-innodb
+
+[mysqld.3]
+log-slave-updates
+loose-innodb
+
+[ENV]
+SERVER_MYPORT_3= @mysqld.3.port
+SERVER_MYSOCK_3= @mysqld.3.socket
diff --git a/mysql-test/suite/rpl/t/rpl_mdev33798.test b/mysql-test/suite/rpl/t/rpl_mdev33798.test
new file mode 100644
index 00000000000..1448ed91133
--- /dev/null
+++ b/mysql-test/suite/rpl/t/rpl_mdev33798.test
@@ -0,0 +1,182 @@
+--source include/have_innodb.inc
+--source include/have_log_bin.inc
+--let $rpl_topology=1->2,1->3
+--source include/rpl_init.inc
+--connect (server_2b,127.0.0.1,root,,,$SERVER_MYPORT_2)
+
+--connection server_2
+SET @old_parallel_threads=@@GLOBAL.slave_parallel_threads;
+SET @old_parallel_mode= @@GLOBAL.slave_parallel_mode;
+SET @old_timeout= @@GLOBAL.lock_wait_timeout;
+SET @old_innodb_timeout= @@GLOBAL.innodb_lock_wait_timeout;
+--source include/stop_slave.inc
+SET GLOBAL slave_parallel_threads=5;
+set global slave_parallel_mode= aggressive;
+# High timeout so we get replication sync error and test failure if the
+# conflict handling is insufficient and lock wait timeout occurs.
+SET GLOBAL lock_wait_timeout= 86400;
+SET GLOBAL innodb_lock_wait_timeout= 86400;
+SET STATEMENT sql_log_bin=0 FOR ALTER TABLE mysql.gtid_slave_pos ENGINE=InnoDB;
+--source include/start_slave.inc
+
+--connection server_1
+CREATE TABLE t1 (a INT PRIMARY KEY, b INT) ENGINE=InnoDB;
+INSERT INTO t1 VALUES (1, 0), (2, 0), (3, 0), (4, 0), (5, 0), (6, 0), (7, 0), (8, 0);
+--save_master_pos
+
+--connection server_2
+--sync_with_master
+--source include/stop_slave.inc
+
+# Test the following scenario:
+#
+# Transactions T1, T2 in domain 1, U1, U2 in domain 2.
+# Wait cycle T1->U2->U1->T2->T1 as follows:
+# T1 row lock wait on U2
+# U2 wait_for_prior_commit on U1
+# U1 row lock wait on T2
+# T2 wait_for_prior_commit on T1
+#
+# Test that the wait cycle is broken correctly with deadlock kill.
+
+--connection server_2b
+# Temporarily block T1 and U1.
+BEGIN;
+SELECT * FROM t1 WHERE a=1 FOR UPDATE;
+SELECT * FROM t1 WHERE a=5 FOR UPDATE;
+
+--connection server_1
+
+SET SESSION gtid_domain_id= 1;
+# T1 in domain 1
+BEGIN;
+UPDATE t1 SET b=1 WHERE a=1;
+UPDATE t1 SET b=1 WHERE a=7;
+COMMIT;
+# T2 in domain 1
+UPDATE t1 SET b=2 WHERE a=3;
+
+SET SESSION gtid_domain_id=2;
+# U1 in domain 2
+BEGIN;
+UPDATE t1 SET b=3 WHERE a=5;
+UPDATE t1 SET b=3 WHERE a=3;
+COMMIT;
+# U2 in domain 2
+UPDATE t1 SET b=4 WHERE a=7;
+SET SESSION gtid_domain_id= 0;
+--source include/save_master_gtid.inc
+
+--connection server_2
+--source include/start_slave.inc
+# Wait until T2, U2 are holding the row locks.
+--let $wait_condition= SELECT COUNT(*)=2 FROM INFORMATION_SCHEMA.PROCESSLIST WHERE state LIKE '%Waiting for prior transaction to commit%'
+--source include/wait_condition.inc
+
+# Then let T1, U1 continue to conflict on the row locks, and check that
+# replication correctly handles the conflict.
+--connection server_2b
+ROLLBACK;
+
+--connection server_2
+--source include/sync_with_master_gtid.inc
+
+# Allow either domain to "win" on the conflicting updates.
+SELECT a, (
+ (a=1 AND b=1) OR
+ (a=3 AND (b=2 OR b=3)) OR
+ (a=5 AND b=3) OR
+ (a=7 AND (b=1 OR b=4)) OR
+ ((a MOD 2)=0 AND b=0)) AS `ok`
+ FROM t1
+ ORDER BY a;
+
+# Now try the same thing with multi-source replication.
+
+# Make server_3 a second master
+--connection server_3
+--source include/sync_with_master_gtid.inc
+--source include/stop_slave.inc
+
+--connection server_2
+--source include/stop_slave.inc
+--replace_result $SERVER_MYPORT_3 MYPORT_3
+eval CHANGE MASTER 'm2' to master_port=$SERVER_MYPORT_3 , master_host='127.0.0.1', master_user='root', master_use_gtid=slave_pos;
+
+--connection server_1
+
+SET SESSION gtid_domain_id= 1;
+# T1 in domain 1
+BEGIN;
+UPDATE t1 SET b=11 WHERE a=1;
+UPDATE t1 SET b=11 WHERE a=7;
+COMMIT;
+# T2 in domain 1
+UPDATE t1 SET b=12 WHERE a=3;
+SET SESSION gtid_domain_id= 1;
+
+--connection server_3
+SET SESSION gtid_domain_id=3;
+# U1 in domain 3
+BEGIN;
+UPDATE t1 SET b=13 WHERE a=5;
+UPDATE t1 SET b=13 WHERE a=3;
+COMMIT;
+# U2 in domain 3
+UPDATE t1 SET b=14 WHERE a=7;
+--source include/save_master_gtid.inc
+
+--connection server_2b
+# Temporarily block T1 and U1.
+BEGIN;
+SELECT * FROM t1 WHERE a=1 FOR UPDATE;
+SELECT * FROM t1 WHERE a=5 FOR UPDATE;
+
+START ALL SLAVES;
+# Wait until T2, U2 are holding the row locks.
+--let $wait_condition= SELECT COUNT(*)=2 FROM INFORMATION_SCHEMA.PROCESSLIST WHERE state LIKE '%Waiting for prior transaction to commit%'
+--source include/wait_condition.inc
+
+--connection server_2b
+ROLLBACK;
+
+--connection server_1
+--source include/save_master_gtid.inc
+--connection server_2
+--source include/sync_with_master_gtid.inc
+--connection server_3
+--source include/save_master_gtid.inc
+--connection server_2
+--source include/sync_with_master_gtid.inc
+
+SELECT a, (
+ (a=1 AND b=11) OR
+ (a=3 AND (b=12 OR b=13)) OR
+ (a=5 AND b=13) OR
+ (a=7 AND (b=11 OR b=14)) OR
+ ((a MOD 2)=0 AND b=0)) AS `ok`
+ FROM t1
+ ORDER BY a;
+
+SET default_master_connection = 'm2';
+--source include/stop_slave.inc
+RESET SLAVE 'm2' ALL;
+SET default_master_connection = '';
+
+--connection server_3
+--source include/start_slave.inc
+
+# Cleanup
+
+--disconnect server_2b
+--connection server_1
+DROP TABLE t1;
+--connection server_2
+--source include/stop_slave.inc
+SET GLOBAL slave_parallel_threads=@old_parallel_threads;
+set global slave_parallel_mode= @old_parallel_mode;
+SET GLOBAL lock_wait_timeout= @old_timeout;
+SET GLOBAL innodb_lock_wait_timeout= @old_innodb_timeout;
+--source include/start_slave.inc
+
+--source include/rpl_end.inc
diff --git a/sql/sql_class.cc b/sql/sql_class.cc
index 0d39f359a09..6026e1e13bb 100644
--- a/sql/sql_class.cc
+++ b/sql/sql_class.cc
@@ -5384,14 +5384,40 @@ thd_rpl_deadlock_check(MYSQL_THD thd, MYSQL_THD other_thd)
return 0;
if (!rgi->is_parallel_exec)
return 0;
- if (rgi->rli != other_rgi->rli)
- return 0;
- if (!rgi->gtid_sub_id || !other_rgi->gtid_sub_id)
- return 0;
- if (rgi->current_gtid.domain_id != other_rgi->current_gtid.domain_id)
- return 0;
- if (rgi->gtid_sub_id > other_rgi->gtid_sub_id)
- return 0;
+ if (rgi->rli == other_rgi->rli)
+ {
+ /*
+ Within the same master connection, we can compare transaction order on
+ the GTID sub_id, and rollback the later transaction to allow the earlier
+ transaction to commit first.
+ */
+ if (!rgi->gtid_sub_id || !other_rgi->gtid_sub_id)
+ return 0;
+ if (rgi->current_gtid.domain_id != other_rgi->current_gtid.domain_id &&
+ other_rgi->speculation != rpl_group_info::SPECULATE_OPTIMISTIC)
+ return 0;
+ if (rgi->gtid_sub_id > other_rgi->gtid_sub_id)
+ return 0;
+ }
+ else
+ {
+ /*
+ Lock conflicts between different master connection should usually not
+ occur, but could still happen if user is running some special setup that
+ tolerates conflicting updates (or in case of user error). We do not have a
+ pre-defined ordering of transactions in this case, but we still need to
+ handle conflicts in _some_ way to avoid undetected deadlocks and hangs.
+
+ We do this by rolling back and retrying any transaction that is being
+ _optimistically_ applied. This can be overly conservative in some cases,
+ but should be fine as conflicts between different master connections are
+ not expected to be common. And it ensures that we won't end up in a
+ deadlock and hang due to a transaction doing wait_for_prior_commit while
+ holding locks that block something in another master connection.
+ */
+ if (other_rgi->speculation != rpl_group_info::SPECULATE_OPTIMISTIC)
+ return 0;
+ }
if (rgi->finish_event_group_called || other_rgi->finish_event_group_called)
{
/*
--
2.30.2
1
0

23 Apr '24
The test could fail with a duplicate key error because switching to non-GTID
mode could start at the wrong old-style position. The position could be
wrong when the previous GTID connect was stopped before receiving the fake
GTID list event which gives the old-style position corresponding to the GTID
connected position.
Work-around by injecting an extra event and syncing the slave before
switching to non-GTID mode.
Signed-off-by: Kristian Nielsen <knielsen(a)knielsen-hq.org>
---
.../suite/rpl/r/rpl_gtid_stop_start.result | 4 ++++
mysql-test/suite/rpl/t/rpl_gtid_stop_start.test | 16 ++++++++++++++++
2 files changed, 20 insertions(+)
diff --git a/mysql-test/suite/rpl/r/rpl_gtid_stop_start.result b/mysql-test/suite/rpl/r/rpl_gtid_stop_start.result
index ae0050c353a..e8633cd45bb 100644
--- a/mysql-test/suite/rpl/r/rpl_gtid_stop_start.result
+++ b/mysql-test/suite/rpl/r/rpl_gtid_stop_start.result
@@ -111,6 +111,10 @@ a
6
7
*** MDEV-4486: Allow to start old-style replication even if mysql.gtid_slave_pos is unavailable
+connection server_1;
+INSERT INTO t1 VALUES (8);
+DELETE FROM t1 WHERE a=8;
+connection server_2;
connection server_2;
include/stop_slave.inc
CHANGE MASTER TO master_use_gtid= no;
diff --git a/mysql-test/suite/rpl/t/rpl_gtid_stop_start.test b/mysql-test/suite/rpl/t/rpl_gtid_stop_start.test
index b5ff294908b..032ebb77d1e 100644
--- a/mysql-test/suite/rpl/t/rpl_gtid_stop_start.test
+++ b/mysql-test/suite/rpl/t/rpl_gtid_stop_start.test
@@ -173,6 +173,22 @@ SELECT * FROM t1 ORDER BY a;
--echo *** MDEV-4486: Allow to start old-style replication even if mysql.gtid_slave_pos is unavailable
+# In GTID mode, the old-style replication position is also updated. But during
+# GTID connect, the old-style position is not known until receiving the fake
+# GTID list event, which contains the required position value. If we happened
+# to stop the slave above before this fake GTID list event, the test could fail
+# with duplicate key errors due to switching to non-GTID mode at a wrong
+# position too far back in the binlog.
+#
+# Work-around this by injecting an extra dummt event and syncing the slave to
+# it, ensuring the old-style position will be updated.
+--connection server_1
+INSERT INTO t1 VALUES (8);
+DELETE FROM t1 WHERE a=8;
+--save_master_pos
+--connection server_2
+--sync_with_master
+
--connection server_2
--source include/stop_slave.inc
CHANGE MASTER TO master_use_gtid= no;
--
2.30.2
1
0

[PATCH] Fix "Assertion `THR_PFS_initialized' failed" in main.bootstrap
by Kristian Nielsen 15 Mar '24
by Kristian Nielsen 15 Mar '24
15 Mar '24
The problem is that the manager thread may not have yet started when
bootstrap exits. This causes assertion when the thread tries to access
performance schema data after shutdown_performance_schema() has been called.
Fix by waiting for the manager thread to have started before proceeding.
Signed-off-by: Kristian Nielsen <knielsen(a)knielsen-hq.org>
---
sql/sql_manager.cc | 11 ++++++++++-
1 file changed, 10 insertions(+), 1 deletion(-)
diff --git a/sql/sql_manager.cc b/sql/sql_manager.cc
index 3d3728b9e00..d4a410ce5fb 100644
--- a/sql/sql_manager.cc
+++ b/sql/sql_manager.cc
@@ -76,6 +76,8 @@ pthread_handler_t handle_manager(void *arg __attribute__((unused)))
pthread_detach_this_thread();
manager_thread = pthread_self();
mysql_mutex_lock(&LOCK_manager);
+ manager_thread_in_use = 1;
+ mysql_cond_signal(&COND_manager);
while (!abort_manager)
{
/* XXX: This will need to be made more general to handle different
@@ -135,12 +137,19 @@ void start_handle_manager()
pthread_t hThread;
int err;
DBUG_EXECUTE_IF("delay_start_handle_manager", my_sleep(1000););
- manager_thread_in_use = 1;
mysql_cond_init(key_COND_manager, &COND_manager,NULL);
mysql_mutex_init(key_LOCK_manager, &LOCK_manager, NULL);
if ((err= mysql_thread_create(key_thread_handle_manager, &hThread,
&connection_attrib, handle_manager, 0)))
+ {
sql_print_warning("Can't create handle_manager thread (errno: %M)", err);
+ DBUG_VOID_RETURN;
+ }
+
+ mysql_mutex_lock(&LOCK_manager);
+ while (!manager_thread_in_use)
+ mysql_cond_wait(&COND_manager, &LOCK_manager);
+ mysql_mutex_unlock(&LOCK_manager);
}
DBUG_VOID_RETURN;
}
--
2.30.2
1
0

[PATCH] MDEV-33551: Semi-sync Wait Point AFTER_COMMIT Slow on Workloads with Heavy Concurrency
by Kristian Nielsen 12 Mar '24
by Kristian Nielsen 12 Mar '24
12 Mar '24
From: Brandon Nesterenko <brandon.nesterenko(a)mariadb.com>
When using semi-sync replication with
rpl_semi_sync_master_wait_point=AFTER_COMMIT, the performance of the
primary can significantly reduce compared to AFTER_SYNC's
performance for workloads with many concurrent users executing
transactions. This is because all connections on the primary share
the same cond_wait variable/mutex pair, so any time an ACK is
received from a replica, all waiting connections are awoken to check
if the ACK was for itself, which is done in mutual exclusion.
This patch changes this such that the waiting THD will use its own
local condition variable, and the ACK receiver thread only signals
connections which have been ACKed for wakeup. That is, the
THD::LOCK_wakeup_ready condition variable is re-used for this
purpose, and the Active_tranx queue nodes are extended to hold the
waiting thread, so it can be signalled once ACKed.
Additionally:
1) At master shutdown (when waiting for slaves), instead of the
main loop individually waiting for each ACK,
await_slave_reply() (renamed await_all_slave_replies) now
awaits ACKs for each transaction within one invocation.
2) The time when thd::is_awaiting_semi_sync_ack is set is moved
to at binlogging time, to ensure transactions which have been
binlogged and queued up to await an ACK are not killed,
and are still waited on.
Review note: DBUG_ASSERT addition from previous commit is changed
to a print warning, and the .test assert_greps to ensure its
absence.
Reviewed By:
============
Kristian Nielsen <knielsen(a)knielsen-hq.org>
---
.../r/rpl_semi_sync_cond_var_per_thd.result | 32 +++
.../rpl/t/rpl_semi_sync_cond_var_per_thd.test | 10 +
sql/log.cc | 12 +-
sql/mysqld.cc | 14 +-
sql/semisync_master.cc | 241 +++++++++++-------
sql/semisync_master.h | 122 +++++++--
sql/sql_class.h | 14 +-
7 files changed, 311 insertions(+), 134 deletions(-)
create mode 100644 mysql-test/suite/rpl/r/rpl_semi_sync_cond_var_per_thd.result
diff --git a/mysql-test/suite/rpl/r/rpl_semi_sync_cond_var_per_thd.result b/mysql-test/suite/rpl/r/rpl_semi_sync_cond_var_per_thd.result
new file mode 100644
index 00000000000..08f601447d5
--- /dev/null
+++ b/mysql-test/suite/rpl/r/rpl_semi_sync_cond_var_per_thd.result
@@ -0,0 +1,32 @@
+include/master-slave.inc
+[connection master]
+connection master;
+call mtr.add_suppression("Got an error reading communication packets");
+set @save_bgc_count= @@global.binlog_commit_wait_count;
+set @save_bgc_usec= @@global.binlog_commit_wait_usec;
+set @save_debug_dbug= @@global.debug_dbug;
+set @@global.binlog_commit_wait_count=3;
+set @@global.binlog_commit_wait_usec=10000000;
+set @@global.debug_dbug="+d,testing_cond_var_per_thd";
+# Ensure semi-sync is on
+connection slave;
+connection master;
+# Create three transactions to binlog group commit together
+connection master;
+create table t1 (a int);
+connection server_1;
+create table t2 (a int);
+connection default;
+create table t3 (a int);
+connection master;
+connection server_1;
+connection default;
+include/assert_grep.inc [Check that there is no 'Thread awaiting semi-sync ACK was awoken before its ACK' warning in error log.]
+#
+# Cleanup
+connection master;
+set @@global.binlog_commit_wait_count=@save_bgc_count;
+set @@global.binlog_commit_wait_usec=@save_bgc_usec;
+set @@global.debug_dbug=@save_debug_dbug;
+drop table t1, t2, t3;
+include/rpl_end.inc
diff --git a/mysql-test/suite/rpl/t/rpl_semi_sync_cond_var_per_thd.test b/mysql-test/suite/rpl/t/rpl_semi_sync_cond_var_per_thd.test
index f8fa0a99d9c..2d9a6d13048 100644
--- a/mysql-test/suite/rpl/t/rpl_semi_sync_cond_var_per_thd.test
+++ b/mysql-test/suite/rpl/t/rpl_semi_sync_cond_var_per_thd.test
@@ -22,10 +22,13 @@
--source include/master-slave.inc
--connection master
+call mtr.add_suppression("Got an error reading communication packets");
set @save_bgc_count= @@global.binlog_commit_wait_count;
set @save_bgc_usec= @@global.binlog_commit_wait_usec;
+set @save_debug_dbug= @@global.debug_dbug;
set @@global.binlog_commit_wait_count=3;
set @@global.binlog_commit_wait_usec=10000000;
+set @@global.debug_dbug="+d,testing_cond_var_per_thd";
--echo # Ensure semi-sync is on
--connection slave
@@ -53,12 +56,19 @@ source include/wait_for_status_var.inc;
--connection default
--reap
+--let $assert_text= Check that there is no 'Thread awaiting semi-sync ACK was awoken before its ACK' warning in error log.
+--let $assert_select=Thread awaiting semi-sync ACK was awoken before its ACK
+--let $assert_file= $MYSQLTEST_VARDIR/log/mysqld.1.err
+--let $assert_count= 0
+--let $assert_only_after=CURRENT_TEST
+--source include/assert_grep.inc
--echo #
--echo # Cleanup
--connection master
set @@global.binlog_commit_wait_count=@save_bgc_count;
set @@global.binlog_commit_wait_usec=@save_bgc_usec;
+set @@global.debug_dbug=@save_debug_dbug;
drop table t1, t2, t3;
--source include/rpl_end.inc
diff --git a/sql/log.cc b/sql/log.cc
index ce5970b6a03..6df6ee7a03c 100644
--- a/sql/log.cc
+++ b/sql/log.cc
@@ -6870,8 +6870,8 @@ bool MYSQL_BIN_LOG::write(Log_event *event_info, my_bool *with_annotate)
mysql_mutex_assert_not_owner(&LOCK_after_binlog_sync);
mysql_mutex_assert_not_owner(&LOCK_commit_ordered);
#ifdef HAVE_REPLICATION
- if (repl_semisync_master.report_binlog_update(thd, log_file_name,
- file->pos_in_file))
+ if (repl_semisync_master.report_binlog_update(
+ thd, thd, log_file_name, file->pos_in_file))
{
sql_print_error("Failed to run 'after_flush' hooks");
error= 1;
@@ -8467,7 +8467,7 @@ MYSQL_BIN_LOG::trx_group_commit_leader(group_commit_entry *leader)
#ifdef HAVE_REPLICATION
if (likely(!current->error) &&
unlikely(repl_semisync_master.
- report_binlog_update(current->thd,
+ report_binlog_update(current->thd, leader->thd,
current->cache_mngr->
last_commit_pos_file,
current->cache_mngr->
@@ -8549,17 +8549,19 @@ MYSQL_BIN_LOG::trx_group_commit_leader(group_commit_entry *leader)
mysql_mutex_assert_not_owner(&LOCK_commit_ordered);
bool first __attribute__((unused))= true;
- bool last __attribute__((unused));
+ bool last;
for (current= queue; current != NULL; current= current->next)
{
last= current->next == NULL;
#ifdef HAVE_REPLICATION
+
if (likely(!current->error))
current->error=
repl_semisync_master.wait_after_sync(current->cache_mngr->
last_commit_pos_file,
current->cache_mngr->
- last_commit_pos_offset);
+ last_commit_pos_offset,
+ last);
#endif
first= false;
}
diff --git a/sql/mysqld.cc b/sql/mysqld.cc
index e224871795e..b315edc091c 100644
--- a/sql/mysqld.cc
+++ b/sql/mysqld.cc
@@ -1750,18 +1750,12 @@ static void close_connections(void)
/*
If we are waiting on any ACKs, delay killing the thread until either an ACK
is received or the timeout is hit.
-
- Allow at max the number of sessions to await a timeout; however, if all
- ACKs have been received in less iterations, then quit early
*/
- if (shutdown_wait_for_slaves && repl_semisync_master.get_master_enabled())
+ if (shutdown_wait_for_slaves && repl_semisync_master.get_master_enabled() &&
+ repl_semisync_master.sync_get_master_wait_sessions())
{
- int waiting_threads= repl_semisync_master.sync_get_master_wait_sessions();
- if (waiting_threads)
- sql_print_information("Delaying shutdown to await semi-sync ACK");
-
- while (waiting_threads-- > 0)
- repl_semisync_master.await_slave_reply();
+ sql_print_information("Delaying shutdown to await semi-sync ACK");
+ repl_semisync_master.await_all_slave_replies();
}
DBUG_EXECUTE_IF("delay_shutdown_phase_2_after_semisync_wait",
diff --git a/sql/semisync_master.cc b/sql/semisync_master.cc
index 0eaf0f0e0e2..411b3dfc84e 100644
--- a/sql/semisync_master.cc
+++ b/sql/semisync_master.cc
@@ -68,6 +68,14 @@ static ulonglong timespec_to_usec(const struct timespec *ts)
return (ulonglong) ts->tv_sec * TIME_MILLION + ts->tv_nsec / TIME_THOUSAND;
}
+int signal_waiting_transaction(THD *waiting_thd, const char *binlog_file,
+ my_off_t binlog_pos)
+{
+ if (waiting_thd->is_awaiting_semisync_ack)
+ mysql_cond_signal(&waiting_thd->COND_wakeup_ready);
+ return 0;
+}
+
/*******************************************************************************
*
* <Active_tranx> class : manage all active transaction nodes
@@ -142,7 +150,18 @@ int Active_tranx::compare(const char *log_file_name1, my_off_t log_file_pos1,
return 0;
}
-int Active_tranx::insert_tranx_node(const char *log_file_name,
+int Active_tranx::for_front(active_tranx_action action)
+{
+ /*
+ This should be called after validating !Active_tranx::is_empty() while
+ holding LOCK_binlog
+ */
+ DBUG_ASSERT(m_trx_front);
+ return action(m_trx_front->thd, m_trx_front->log_name, m_trx_front->log_pos);
+}
+
+int Active_tranx::insert_tranx_node(THD *thd_to_wait,
+ const char *log_file_name,
my_off_t log_file_pos)
{
Tranx_node *ins_node;
@@ -165,6 +184,7 @@ int Active_tranx::insert_tranx_node(const char *log_file_name,
strncpy(ins_node->log_name, log_file_name, FN_REFLEN-1);
ins_node->log_name[FN_REFLEN-1] = 0; /* make sure it ends properly */
ins_node->log_pos = log_file_pos;
+ ins_node->thd= thd_to_wait;
if (!m_trx_front)
{
@@ -232,28 +252,22 @@ bool Active_tranx::is_tranx_end_pos(const char *log_file_name,
DBUG_RETURN(entry != NULL);
}
-void Active_tranx::clear_active_tranx_nodes(const char *log_file_name,
- my_off_t log_file_pos)
+void Active_tranx::clear_active_tranx_nodes(
+ const char *log_file_name, my_off_t log_file_pos,
+ active_tranx_action pre_delete_hook)
{
Tranx_node *new_front;
DBUG_ENTER("Active_tranx::::clear_active_tranx_nodes");
- if (log_file_name != NULL)
+ new_front= m_trx_front;
+ while (new_front)
{
- new_front = m_trx_front;
-
- while (new_front)
- {
- if (compare(new_front, log_file_name, log_file_pos) > 0)
- break;
- new_front = new_front->next;
- }
- }
- else
- {
- /* If log_file_name is NULL, clear everything. */
- new_front = NULL;
+ if ((log_file_name != NULL) &&
+ compare(new_front, log_file_name, log_file_pos) > 0)
+ break;
+ pre_delete_hook(new_front->thd, new_front->log_name, new_front->log_pos);
+ new_front = new_front->next;
}
if (new_front == NULL)
@@ -500,19 +514,31 @@ void Repl_semi_sync_master::unlock()
mysql_mutex_unlock(&LOCK_binlog);
}
-void Repl_semi_sync_master::cond_broadcast()
-{
- mysql_cond_broadcast(&COND_binlog_send);
-}
-
-int Repl_semi_sync_master::cond_timewait(struct timespec *wait_time)
+int Repl_semi_sync_master::cond_timewait(THD *thd, struct timespec *_wait_time)
{
int wait_res;
+ struct timespec gen_wait_time;
+ struct timespec *real_wait_time;
DBUG_ENTER("Repl_semi_sync_master::cond_timewait()");
- wait_res= mysql_cond_timedwait(&COND_binlog_send,
- &LOCK_binlog, wait_time);
+ if (_wait_time == NULL)
+ {
+ create_timeout(&gen_wait_time, NULL);
+ real_wait_time= &gen_wait_time;
+ }
+ else
+ {
+ real_wait_time= _wait_time;
+ }
+
+ /*
+ All connection threads share the mutex (LOCK_binlog) to keep consistent
+ with the Active_tranx cache, but each thread has its own condition variable
+ on which it waits.
+ */
+ wait_res= mysql_cond_timedwait(&thd->COND_wakeup_ready, &LOCK_binlog,
+ real_wait_time);
DBUG_RETURN(wait_res);
}
@@ -533,7 +559,8 @@ void Repl_semi_sync_master::remove_slave()
Signal transactions waiting in commit_trx() that they do not have to
wait anymore.
*/
- cond_broadcast();
+ m_active_tranxs->clear_active_tranx_nodes(NULL, NULL,
+ signal_waiting_transaction);
}
unlock();
}
@@ -616,7 +643,6 @@ int Repl_semi_sync_master::report_reply_binlog(uint32 server_id,
my_off_t log_file_pos)
{
int cmp;
- bool can_release_threads = false;
bool need_copy_send_pos = true;
DBUG_ENTER("Repl_semi_sync_master::report_reply_binlog");
@@ -668,45 +694,26 @@ int Repl_semi_sync_master::report_reply_binlog(uint32 server_id,
/* Remove all active transaction nodes before this point. */
DBUG_ASSERT(m_active_tranxs != NULL);
- m_active_tranxs->clear_active_tranx_nodes(log_file_name, log_file_pos);
+ m_active_tranxs->clear_active_tranx_nodes(log_file_name, log_file_pos,
+ signal_waiting_transaction);
+ m_wait_file_name_inited= false;
DBUG_PRINT("semisync", ("%s: Got reply at (%s, %lu)",
"Repl_semi_sync_master::report_reply_binlog",
log_file_name, (ulong)log_file_pos));
}
- if (rpl_semi_sync_master_wait_sessions > 0)
- {
- /* Let us check if some of the waiting threads doing a trx
- * commit can now proceed.
- */
- cmp = Active_tranx::compare(m_reply_file_name, m_reply_file_pos,
- m_wait_file_name, m_wait_file_pos);
- if (cmp >= 0)
- {
- /* Yes, at least one waiting thread can now proceed:
- * let us release all waiting threads with a broadcast
- */
- can_release_threads = true;
- m_wait_file_name_inited = false;
- }
- }
l_end:
unlock();
- if (can_release_threads)
- {
- DBUG_PRINT("semisync", ("%s: signal all waiting threads.",
- "Repl_semi_sync_master::report_reply_binlog"));
-
- cond_broadcast();
- }
DBUG_RETURN(0);
}
-int Repl_semi_sync_master::wait_after_sync(const char *log_file, my_off_t log_pos)
+int Repl_semi_sync_master::wait_after_sync(const char *log_file,
+ my_off_t log_pos,
+ bool unmark_thd_awaiting_ack)
{
if (!get_master_enabled())
return 0;
@@ -714,7 +721,8 @@ int Repl_semi_sync_master::wait_after_sync(const char *log_file, my_off_t log_po
int ret= 0;
if(log_pos &&
wait_point() == SEMI_SYNC_MASTER_WAIT_POINT_AFTER_BINLOG_SYNC)
- ret= commit_trx(log_file + dirname_length(log_file), log_pos);
+ ret= commit_trx(log_file + dirname_length(log_file), log_pos,
+ unmark_thd_awaiting_ack);
return ret;
}
@@ -762,24 +770,27 @@ int Repl_semi_sync_master::wait_after_rollback(THD *thd, bool all)
/**
The method runs after flush to binary log is done.
*/
-int Repl_semi_sync_master::report_binlog_update(THD* thd, const char *log_file,
+int Repl_semi_sync_master::report_binlog_update(THD *trans_thd,
+ THD *waiter_thd,
+ const char *log_file,
my_off_t log_pos)
{
if (get_master_enabled())
{
Trans_binlog_info *log_info;
- if (!(log_info= thd->semisync_info))
+ if (!(log_info= trans_thd->semisync_info))
{
if(!(log_info= (Trans_binlog_info*)my_malloc(PSI_INSTRUMENT_ME,
sizeof(Trans_binlog_info), MYF(0))))
return 1;
- thd->semisync_info= log_info;
+ trans_thd->semisync_info= log_info;
}
strcpy(log_info->log_file, log_file + dirname_length(log_file));
log_info->log_pos = log_pos;
- return write_tranx_in_binlog(log_info->log_file, log_pos);
+ return write_tranx_in_binlog(waiter_thd, log_info->log_file,
+ log_pos);
}
return 0;
@@ -825,8 +836,9 @@ void Repl_semi_sync_master::dump_end(THD* thd)
ack_receiver.remove_slave(thd);
}
-int Repl_semi_sync_master::commit_trx(const char* trx_wait_binlog_name,
- my_off_t trx_wait_binlog_pos)
+int Repl_semi_sync_master::commit_trx(const char *trx_wait_binlog_name,
+ my_off_t trx_wait_binlog_pos,
+ bool unmark_thd_awaiting_ack)
{
bool success= 0;
DBUG_ENTER("Repl_semi_sync_master::commit_trx");
@@ -852,9 +864,8 @@ int Repl_semi_sync_master::commit_trx(const char* trx_wait_binlog_name,
lock();
/* This must be called after acquired the lock */
- THD_ENTER_COND(thd, &COND_binlog_send, &LOCK_binlog,
- & stage_waiting_for_semi_sync_ack_from_slave,
- & old_stage);
+ THD_ENTER_COND(thd, &thd->COND_wakeup_ready, &LOCK_binlog,
+ &stage_waiting_for_semi_sync_ack_from_slave, &old_stage);
/* This is the real check inside the mutex. */
if (!get_master_enabled() || !is_on())
@@ -902,7 +913,7 @@ int Repl_semi_sync_master::commit_trx(const char* trx_wait_binlog_name,
trx_wait_binlog_pos,
m_wait_file_name, m_wait_file_pos);
if (cmp <= 0)
- {
+ {
/* This thd has a lower position, let's update the minimum info. */
strmake_buf(m_wait_file_name, trx_wait_binlog_name);
m_wait_file_pos = trx_wait_binlog_pos;
@@ -934,20 +945,13 @@ int Repl_semi_sync_master::commit_trx(const char* trx_wait_binlog_name,
*/
rpl_semi_sync_master_wait_sessions++;
- /* We keep track of when this thread is awaiting an ack to ensure it is
- * not killed while awaiting an ACK if a shutdown is issued.
- */
- set_thd_awaiting_semisync_ack(thd, TRUE);
-
DBUG_PRINT("semisync", ("%s: wait %lu ms for binlog sent (%s, %lu)",
"Repl_semi_sync_master::commit_trx",
m_wait_timeout,
m_wait_file_name, (ulong)m_wait_file_pos));
create_timeout(&abstime, &start_ts);
- wait_result = cond_timewait(&abstime);
-
- set_thd_awaiting_semisync_ack(thd, FALSE);
+ wait_result = cond_timewait(thd, &abstime);
rpl_semi_sync_master_wait_sessions--;
if (wait_result != 0)
@@ -979,14 +983,29 @@ int Repl_semi_sync_master::commit_trx(const char* trx_wait_binlog_name,
{
rpl_semi_sync_master_trx_wait_num++;
rpl_semi_sync_master_trx_wait_time += wait_time;
- /*
- Assert we have either recieved our ACK; or have timed out and are
- awoken in an off state.
- */
- DBUG_ASSERT(!get_master_enabled() || !is_on() || thd->is_killed() ||
- 0 <= Active_tranx::compare(
- m_reply_file_name, m_reply_file_pos,
- trx_wait_binlog_name, trx_wait_binlog_pos));
+
+ DBUG_EXECUTE_IF("testing_cond_var_per_thd", {
+ /*
+ DBUG log warning to ensure we have either recieved our ACK; or
+ have timed out and are awoken in an off state. Test
+ rpl.rpl_semi_sync_cond_var_per_thd scans the logs to ensure this
+ warning is not present.
+ */
+ bool valid_wakeup=
+ (!get_master_enabled() || !is_on() || thd->is_killed() ||
+ 0 <= Active_tranx::compare(
+ m_reply_file_name, m_reply_file_pos,
+ trx_wait_binlog_name, trx_wait_binlog_pos));
+ if (!valid_wakeup)
+ {
+ sql_print_warning(
+ "Thread awaiting semi-sync ACK was awoken before its "
+ "ACK. THD (%llu), Wait coord: (%s, %llu), ACK coord: (%s, "
+ "%llu)",
+ thd->thread_id, trx_wait_binlog_name, trx_wait_binlog_pos,
+ m_reply_file_name, m_reply_file_pos);
+ }
+ });
}
}
}
@@ -997,7 +1016,8 @@ int Repl_semi_sync_master::commit_trx(const char* trx_wait_binlog_name,
m_active_tranxs may be NULL if someone disabled semi sync during
cond_timewait()
*/
- DBUG_ASSERT(thd_killed(thd) || !m_active_tranxs || aborted ||
+ DBUG_ASSERT(thd_killed(thd) || !m_active_tranxs ||
+ m_active_tranxs->is_empty() || aborted ||
!m_active_tranxs->is_tranx_end_pos(trx_wait_binlog_name,
trx_wait_binlog_pos));
@@ -1008,6 +1028,9 @@ int Repl_semi_sync_master::commit_trx(const char* trx_wait_binlog_name,
else
rpl_semi_sync_master_no_transactions++;
+ if (unmark_thd_awaiting_ack)
+ set_thd_awaiting_semisync_ack(thd, FALSE);
+
/* The lock held will be released by thd_exit_cond, so no need to
call unlock() here */
THD_EXIT_COND(thd, &old_stage);
@@ -1038,20 +1061,22 @@ void Repl_semi_sync_master::switch_off()
{
DBUG_ENTER("Repl_semi_sync_master::switch_off");
+ /* Clear the active transaction list. */
+ if (m_active_tranxs)
+ {
+ m_active_tranxs->clear_active_tranx_nodes(NULL, 0,
+ signal_waiting_transaction);
+ }
if (m_state)
{
m_state = false;
- /* Clear the active transaction list. */
- DBUG_ASSERT(m_active_tranxs != NULL);
- m_active_tranxs->clear_active_tranx_nodes(NULL, 0);
rpl_semi_sync_master_off_times++;
m_wait_file_name_inited = false;
m_reply_file_name_inited = false;
sql_print_information("Semi-sync replication switched OFF.");
}
- cond_broadcast(); /* wake up all waiting threads */
DBUG_VOID_RETURN;
}
@@ -1198,7 +1223,8 @@ int Repl_semi_sync_master::update_sync_header(THD* thd, unsigned char *packet,
DBUG_RETURN(0);
}
-int Repl_semi_sync_master::write_tranx_in_binlog(const char* log_file_name,
+int Repl_semi_sync_master::write_tranx_in_binlog(THD *thd,
+ const char *log_file_name,
my_off_t log_file_pos)
{
int result = 0;
@@ -1241,7 +1267,7 @@ int Repl_semi_sync_master::write_tranx_in_binlog(const char* log_file_name,
if (is_on())
{
DBUG_ASSERT(m_active_tranxs != NULL);
- if(m_active_tranxs->insert_tranx_node(log_file_name, log_file_pos))
+ if(m_active_tranxs->insert_tranx_node(thd, log_file_name, log_file_pos))
{
/*
if insert tranx_node failed, print a warning message
@@ -1253,6 +1279,7 @@ int Repl_semi_sync_master::write_tranx_in_binlog(const char* log_file_name,
}
else
{
+ set_thd_awaiting_semisync_ack(thd, TRUE);
rpl_semi_sync_master_request_ack++;
}
}
@@ -1370,22 +1397,40 @@ void Repl_semi_sync_master::set_export_stats()
unlock();
}
-void Repl_semi_sync_master::await_slave_reply()
+void Repl_semi_sync_master::await_all_slave_replies()
{
- struct timespec abstime;
+ int wait_result;
- DBUG_ENTER("Repl_semi_sync_master::::await_slave_reply");
- lock();
-
- /* Just return if there is nothing to wait for */
- if (!rpl_semi_sync_master_wait_sessions)
- goto end;
+ DBUG_ENTER("Repl_semi_sync_master::::await_all_slave_replies");
- create_timeout(&abstime, NULL);
- cond_timewait(&abstime);
+ /*
+ Wait for all transactions that need ACKS to have received them; or timeout.
+ If it is a timeout, the connection thread should attempt to turn off
+ semi-sync and broadcast to all other waiting threads to move on.
-end:
- unlock();
+ The lock is taken per iteration rather than over the whole loop to allow
+ more opportunity for the user threads to handle their ACKs.
+ */
+ while (TRUE)
+ {
+ lock();
+ if (m_active_tranxs->is_empty() || !get_master_enabled() || !is_on())
+ {
+ unlock();
+ break;
+ }
+ wait_result= m_active_tranxs->for_front(
+ [](THD *thd, const char *binlog_file, my_off_t binlog_pos) -> int {
+ return (thd->is_awaiting_semisync_ack)
+ ? repl_semisync_master.cond_timewait(thd, NULL)
+ : 0;
+ });
+ unlock();
+
+ // Check for timeout
+ if (wait_result != 0)
+ break;
+ }
DBUG_VOID_RETURN;
}
diff --git a/sql/semisync_master.h b/sql/semisync_master.h
index 99f46869354..1eb690b572b 100644
--- a/sql/semisync_master.h
+++ b/sql/semisync_master.h
@@ -31,6 +31,7 @@ extern PSI_cond_key key_COND_binlog_send;
struct Tranx_node {
char log_name[FN_REFLEN];
my_off_t log_pos;
+ THD *thd; /* The thread awaiting an ACK */
struct Tranx_node *next; /* the next node in the sorted list */
struct Tranx_node *hash_next; /* the next node during hash collision */
};
@@ -288,6 +289,18 @@ class Tranx_node_allocator
}
};
+/**
+ Function pointer type to run on the contents of an Active_tranx node.
+
+ Return 0 for success, 1 for error.
+
+ Note Repl_semi_sync_master::LOCK_binlog is not guaranteed to be held for
+ its invocation. See the context in which it is called to know.
+*/
+
+typedef int (*active_tranx_action)(THD *trx_thd, const char *log_file_name,
+ my_off_t trx_log_file_pos);
+
/**
This class manages memory for active transaction list.
@@ -338,29 +351,60 @@ class Active_tranx
* Return:
* 0: success; non-zero: error
*/
- int insert_tranx_node(const char *log_file_name, my_off_t log_file_pos);
+ int insert_tranx_node(THD *thd_to_wait, const char *log_file_name,
+ my_off_t log_file_pos);
/* Clear the active transaction nodes until(inclusive) the specified
* position.
* If log_file_name is NULL, everything will be cleared: the sorted
* list and the hash table will be reset to empty.
+ *
+ * The pre_delete_hook parameter is a function pointer that will be invoked
+ * for each Active_tranx node, in order, from m_trx_front to log_file_name,
+ * e.g. to signal their wakeup condition. Repl_semi_sync_binlog::LOCK_binlog
+ * is held while this is invoked.
*/
void clear_active_tranx_nodes(const char *log_file_name,
- my_off_t log_file_pos);
+ my_off_t log_file_pos,
+ active_tranx_action pre_delete_hook);
/* Given a position, check to see whether the position is an active
* transaction's ending position by probing the hash table.
*/
bool is_tranx_end_pos(const char *log_file_name, my_off_t log_file_pos);
+ /* Invoke action (function parameter) on only the first element in the list
+ * of active transactions.
+ *
+ * Note this function does not explicitly lock
+ * Repl_semi_sync_master::LOCK_binlog.
+ */
+ int for_front(active_tranx_action action);
+
/* Given two binlog positions, compare which one is bigger based on
* (file_name, file_position).
*/
static int compare(const char *log_file_name1, my_off_t log_file_pos1,
const char *log_file_name2, my_off_t log_file_pos2);
+
+ /* Check if there are no transactions actively awaiting ACKs. Returns true
+ * if the internal linked list has no entries, false otherwise.
+ */
+ bool is_empty() { return m_trx_front == NULL; }
+
};
+/*
+ Element in Repl_semi_sync_master::wait_queue to preserve the state of a
+ transaction waiting for an ACK.
+*/
+typedef struct _semisync_wait_trx {
+ const char *binlog_name;
+ my_off_t binlog_pos;
+ THD *thd;
+} semisync_wait_trx_t;
+
/**
The extension class for the master of semi-synchronous replication
*/
@@ -433,8 +477,11 @@ class Repl_semi_sync_master
void lock();
void unlock();
- void cond_broadcast();
- int cond_timewait(struct timespec *wait_time);
+
+ /* Do a cond wait on thd->LOCK_wakup_ready. If wait_time is NULL, timeout
+ * will use m_wait_timeout.
+ */
+ int cond_timewait(THD *thd, struct timespec *wait_time);
/* Is semi-sync replication on? */
bool is_on() {
@@ -482,10 +529,11 @@ class Repl_semi_sync_master
void create_timeout(struct timespec *out, struct timespec *start_arg);
/*
- Blocks the calling thread until the ack_receiver either receives an ACK
- or times out (from rpl_semi_sync_master_timeout)
+ Blocks the calling thread until the ack_receiver either receives ACKs for
+ all transactions awaiting ACKs, or times out (from
+ rpl_semi_sync_master_timeout)
*/
- void await_slave_reply();
+ void await_all_slave_replies();
/*set the ACK point, after binlog sync or after transaction commit*/
void set_wait_point(unsigned long ack_point)
@@ -544,26 +592,56 @@ class Repl_semi_sync_master
* all other transaction would not wait either.
*
* Input: (the transaction events' ending binlog position)
- * trx_wait_binlog_name - (IN) ending position's file name
- * trx_wait_binlog_pos - (IN) ending position's file offset
+ * trx_wait_binlog_name - (IN) ending position's file name
+ * trx_wait_binlog_pos - (IN) ending position's file offset
+ * unmark_thd_awaiting_ack - (IN) Whether or not to unmark
+ * thd->is_awaiting_semisync_ack after receiving
+ * an ACK from the replica. If using wait_point
+ * AFTER_SYNC with binlog_group_commit, only the
+ * last transaction written to binlog in the
+ * group should negate the boolean, because the
+ * same thread (i.e. leader thread) will wait for
+ * all transaction ACKs. Negating it too early
+ * would break SHUTDOWN WAIT FOR ALL SLAVES, as
+ * that is the condition tested to bypass killing
+ * threads in phase 1. In all other situations,
+ * the boolean should be negated immediately.
*
* Return:
* 0: success; non-zero: error
*/
int commit_trx(const char* trx_wait_binlog_name,
- my_off_t trx_wait_binlog_pos);
+ my_off_t trx_wait_binlog_pos,
+ bool unmark_thd_awaiting_ack=TRUE);
- /*Wait for ACK after writing/sync binlog to file*/
- int wait_after_sync(const char* log_file, my_off_t log_pos);
+ /* Wait for ACK after writing/sync binlog to file
+ * For details on parameters, see commit_trx() function declaration comment.
+ */
+ int wait_after_sync(const char *log_file, my_off_t log_pos,
+ bool unmark_thd_awaiting_ack= TRUE);
/*Wait for ACK after commting the transaction*/
int wait_after_commit(THD* thd, bool all);
/*Wait after the transaction is rollback*/
int wait_after_rollback(THD *thd, bool all);
- /*Store the current binlog position in m_active_tranxs. This position should
- * be acked by slave*/
- int report_binlog_update(THD *thd, const char *log_file,my_off_t log_pos);
+ /* Store the current binlog position in m_active_tranxs. This position should
+ * be acked by slave.
+ *
+ * Inputs:
+ * trans_thd Thread of the transaction which is executing the
+ * transaction.
+ * waiter_thd Thread that will wait for the ACK from the replica,
+ * which depends on the semi-sync wait point. If AFTER_SYNC,
+ * and also using binlog group commit, this will be the leader
+ * thread of the binlog commit. Otherwise, it is the thread that
+ * is executing the transaction, i.e. the same as trans_thd.
+ * log_file Name of the binlog file that the transaction is written into
+ * log_pos Offset within the binlog file that the transaction is written
+ * at
+ */
+ int report_binlog_update(THD *trans_thd, THD *waiter_thd,
+ const char *log_file, my_off_t log_pos);
int dump_start(THD* thd,
const char *log_file,
@@ -609,13 +687,19 @@ class Repl_semi_sync_master
* semi-sync is on
*
* Input: (the transaction events' ending binlog position)
+ * THD - (IN) thread that will wait for an ACK. This can be the
+ * binlog leader thread when using wait_point
+ * AFTER_SYNC with binlog group commit. In all other
+ * cases, this is the user thread executing the
+ * transaction.
* log_file_name - (IN) transaction ending position's file name
* log_file_pos - (IN) transaction ending position's file offset
*
* Return:
* 0: success; non-zero: error
*/
- int write_tranx_in_binlog(const char* log_file_name, my_off_t log_file_pos);
+ int write_tranx_in_binlog(THD *thd, const char *log_file_name,
+ my_off_t log_file_pos);
/* Read the slave's reply so that we know how much progress the slave makes
* on receive replication events.
@@ -634,9 +718,9 @@ class Repl_semi_sync_master
int before_reset_master();
/*
- Determines if the given thread is currently awaiting a semisync_ack. Note
- that the thread's value is protected by this class's LOCK_binlog, so this
- function (indirectly) provides safe access.
+ Determines if the given thread is currently awaiting (or queued to await) a
+ semisync_ack. Note that the thread's value is protected by this class's
+ LOCK_binlog, so this function (indirectly) provides safe access.
*/
my_bool is_thd_awaiting_semisync_ack(THD *thd)
{
diff --git a/sql/sql_class.h b/sql/sql_class.h
index 4c6005416e1..35d72d61d48 100644
--- a/sql/sql_class.h
+++ b/sql/sql_class.h
@@ -5318,8 +5318,18 @@ class THD: public THD_count, /* this must be first */
Flag, mutex and condition for a thread to wait for a signal from another
thread.
- Currently used to wait for group commit to complete, can also be used for
- other purposes.
+ Currently used to wait for group commit to complete, and COND_wakeup_ready
+ is used for threads to wait on semi-sync ACKs (though is protected by
+ Repl_semi_sync_master::LOCK_binlog). Note the following relationships
+ between these two use-cases when using
+ rpl_semi_sync_master_wait_point=AFTER_SYNC during group commit:
+ 1) Non-leader threads use COND_wakeup_ready to wait for the leader thread
+ to complete binlog commit.
+ 2) The leader thread uses COND_wakeup_ready to await ACKs from the
+ replica before signalling the non-leader threads to wake up.
+
+ With wait_point=AFTER_COMMIT, there is no overlap as binlogging has
+ finished, so COND_wakeup_ready is safe to re-use.
*/
bool wakeup_ready;
mysql_mutex_t LOCK_wakeup_ready;
--
2.30.2
1
0

[PATCH] MDEV-24622: Replication does not support bulk insert into empty table
by Kristian Nielsen 08 Mar '24
by Kristian Nielsen 08 Mar '24
08 Mar '24
Remove work-around that disables bulk insert optimization in replication
The root cause of the original problem is now fixed (MDEV-33475). Though the
bulk insert optimization will still be disabled in replication, as it is
only enabled in special circumstances meant for loading a mysqldump.
Signed-off-by: Kristian Nielsen <knielsen(a)knielsen-hq.org>
---
sql/sql_class.cc | 8 --------
storage/innobase/row/row0ins.cc | 9 +--------
2 files changed, 1 insertion(+), 16 deletions(-)
diff --git a/sql/sql_class.cc b/sql/sql_class.cc
index 0433156ef40..f8ebc513744 100644
--- a/sql/sql_class.cc
+++ b/sql/sql_class.cc
@@ -5317,14 +5317,6 @@ extern "C" enum enum_server_command thd_current_command(MYSQL_THD thd)
return thd->get_command();
}
-#ifdef HAVE_REPLICATION /* Working around MDEV-24622 */
-/** @return whether the current thread is for applying binlog in a replica */
-extern "C" int thd_is_slave(const MYSQL_THD thd)
-{
- return thd && thd->slave_thread;
-}
-#endif /* HAVE_REPLICATION */
-
/* Returns high resolution timestamp for the start
of the current query. */
extern "C" unsigned long long thd_start_utime(const MYSQL_THD thd)
diff --git a/storage/innobase/row/row0ins.cc b/storage/innobase/row/row0ins.cc
index 5d75e88227e..990c14d1a8f 100644
--- a/storage/innobase/row/row0ins.cc
+++ b/storage/innobase/row/row0ins.cc
@@ -2565,12 +2565,6 @@ row_ins_index_entry_big_rec(
return(error);
}
-#ifdef HAVE_REPLICATION /* Working around MDEV-24622 */
-extern "C" int thd_is_slave(const MYSQL_THD thd);
-#else
-# define thd_is_slave(thd) 0
-#endif
-
#if defined __aarch64__&&defined __GNUC__&&__GNUC__==4&&!defined __clang__
/* Avoid GCC 4.8.5 internal compiler error due to srw_mutex::wr_unlock().
We would only need this for row_ins_clust_index_entry_low(),
@@ -2722,8 +2716,7 @@ row_ins_clust_index_entry_low(
&& !index->table->skip_alter_undo
&& !index->table->n_rec_locks
&& !index->table->is_active_ddl()
- && !index->table->versioned()
- && !thd_is_slave(trx->mysql_thd) /* FIXME: MDEV-24622 */) {
+ && !index->table->versioned()) {
DEBUG_SYNC_C("empty_root_page_insert");
if (!index->table->is_temporary()) {
--
2.30.2
1
0

[PATCH] MDEV-33475: --gtid-ignore-duplicate can double-apply event in case of parallel replication retry
by Kristian Nielsen 08 Mar '24
by Kristian Nielsen 08 Mar '24
08 Mar '24
When rolling back and retrying a transaction in parallel replication, don't
release the domain ownership (for --gtid-ignore-duplicates) as part of the
rollback. Otherwise another master connection could grab the ownership and
double-apply the transaction in parallel with the retry.
Signed-off-by: Kristian Nielsen <knielsen(a)knielsen-hq.org>
---
.../gtid_ignore_duplicates.result | 108 +++++++++++++++++-
.../multi_source/gtid_ignore_duplicates.test | 68 ++++++++++-
sql/rpl_parallel.cc | 8 +-
sql/rpl_rli.cc | 4 +-
sql/rpl_rli.h | 2 +-
5 files changed, 178 insertions(+), 12 deletions(-)
diff --git a/mysql-test/suite/multi_source/gtid_ignore_duplicates.result b/mysql-test/suite/multi_source/gtid_ignore_duplicates.result
index e142ff8b981..88b525e21ff 100644
--- a/mysql-test/suite/multi_source/gtid_ignore_duplicates.result
+++ b/mysql-test/suite/multi_source/gtid_ignore_duplicates.result
@@ -174,6 +174,105 @@ a
10
11
12
+*** MDEV-33475: --gtid-ignore-duplicate can double-apply event in case of parallel replication retry
+connection server_2;
+STOP SLAVE "c2b";
+SET default_master_connection = "c2b";
+include/wait_for_slave_to_stop.inc
+STOP SLAVE "a2b";
+SET default_master_connection = "a2b";
+include/wait_for_slave_to_stop.inc
+connection server_1;
+CREATE TABLE t2 (a INT PRIMARY KEY, b INT) ENGINE=InnoDB;
+BEGIN;
+INSERT INTO t2 VALUES (0, 0);
+INSERT INTO t2 VALUES (1, 0);
+INSERT INTO t2 VALUES (2, 0);
+INSERT INTO t2 VALUES (3, 0);
+INSERT INTO t2 VALUES (4, 0);
+INSERT INTO t2 VALUES (5, 0);
+INSERT INTO t2 VALUES (6, 0);
+INSERT INTO t2 VALUES (7, 0);
+INSERT INTO t2 VALUES (8, 0);
+INSERT INTO t2 VALUES (9, 0);
+COMMIT;
+BEGIN;
+INSERT INTO t2 VALUES (0+10, 100);
+UPDATE t2 SET b=0 WHERE a<10;
+INSERT INTO t2 VALUES (0+20, 200);
+COMMIT;
+BEGIN;
+INSERT INTO t2 VALUES (1+10, 100);
+UPDATE t2 SET b=1 WHERE a<10;
+INSERT INTO t2 VALUES (1+20, 200);
+COMMIT;
+BEGIN;
+INSERT INTO t2 VALUES (2+10, 100);
+UPDATE t2 SET b=2 WHERE a<10;
+INSERT INTO t2 VALUES (2+20, 200);
+COMMIT;
+BEGIN;
+INSERT INTO t2 VALUES (3+10, 100);
+UPDATE t2 SET b=3 WHERE a<10;
+INSERT INTO t2 VALUES (3+20, 200);
+COMMIT;
+BEGIN;
+INSERT INTO t2 VALUES (4+10, 100);
+UPDATE t2 SET b=4 WHERE a<10;
+INSERT INTO t2 VALUES (4+20, 200);
+COMMIT;
+BEGIN;
+INSERT INTO t2 VALUES (5+10, 100);
+UPDATE t2 SET b=5 WHERE a<10;
+INSERT INTO t2 VALUES (5+20, 200);
+COMMIT;
+BEGIN;
+INSERT INTO t2 VALUES (6+10, 100);
+UPDATE t2 SET b=6 WHERE a<10;
+INSERT INTO t2 VALUES (6+20, 200);
+COMMIT;
+BEGIN;
+INSERT INTO t2 VALUES (7+10, 100);
+UPDATE t2 SET b=7 WHERE a<10;
+INSERT INTO t2 VALUES (7+20, 200);
+COMMIT;
+BEGIN;
+INSERT INTO t2 VALUES (8+10, 100);
+UPDATE t2 SET b=8 WHERE a<10;
+INSERT INTO t2 VALUES (8+20, 200);
+COMMIT;
+BEGIN;
+INSERT INTO t2 VALUES (9+10, 100);
+UPDATE t2 SET b=9 WHERE a<10;
+INSERT INTO t2 VALUES (9+20, 200);
+COMMIT;
+SELECT COUNT(*), SUM(a), SUM(b) FROM t2;
+COUNT(*) SUM(a) SUM(b)
+30 435 3090
+include/save_master_gtid.inc
+connection server_2;
+SET @old_mode= @@GLOBAL.slave_parallel_mode;
+SET GLOBAL slave_parallel_mode=aggressive;
+SET default_master_connection = "a2b";
+START SLAVE;
+include/wait_for_slave_to_start.inc
+SET default_master_connection = "c2b";
+START SLAVE;
+include/wait_for_slave_to_start.inc
+include/sync_with_master_gtid.inc
+SELECT COUNT(*), SUM(a), SUM(b) FROM t2;
+COUNT(*) SUM(a) SUM(b)
+30 435 3090
+connection server_3;
+include/sync_with_master_gtid.inc
+SELECT COUNT(*), SUM(a), SUM(b) FROM t2;
+COUNT(*) SUM(a) SUM(b)
+30 435 3090
+connection server_4;
+include/sync_with_master_gtid.inc
+SELECT COUNT(*), SUM(a), SUM(b) FROM t2;
+COUNT(*) SUM(a) SUM(b)
+30 435 3090
*** Test also with not using parallel replication.
connection server_1;
SET default_master_connection = "b2a";
@@ -474,6 +573,7 @@ Warnings:
Note 1938 SLAVE 'a2b' stopped
Note 1938 SLAVE 'c2b' stopped
SET GLOBAL slave_parallel_threads= @old_parallel;
+SET GLOBAL slave_parallel_mode= @old_mode;
SET GLOBAL gtid_ignore_duplicates= @old_ignore_duplicates;
connection server_3;
SET GLOBAL gtid_domain_id=0;
@@ -491,22 +591,22 @@ Note 1938 SLAVE 'a2d' stopped
SET GLOBAL slave_parallel_threads= @old_parallel;
SET GLOBAL gtid_ignore_duplicates= @old_ignore_duplicates;
connection server_1;
-DROP TABLE t1;
+DROP TABLE t1, t2;
ALTER TABLE mysql.gtid_slave_pos ENGINE=Aria;
include/reset_master_slave.inc
disconnect server_1;
connection server_2;
-DROP TABLE t1;
+DROP TABLE t1, t2;
ALTER TABLE mysql.gtid_slave_pos ENGINE=Aria;
include/reset_master_slave.inc
disconnect server_2;
connection server_3;
-DROP TABLE t1;
+DROP TABLE t1, t2;
ALTER TABLE mysql.gtid_slave_pos ENGINE=Aria;
include/reset_master_slave.inc
disconnect server_3;
connection server_4;
-DROP TABLE t1;
+DROP TABLE t1, t2;
ALTER TABLE mysql.gtid_slave_pos ENGINE=Aria;
include/reset_master_slave.inc
disconnect server_4;
diff --git a/mysql-test/suite/multi_source/gtid_ignore_duplicates.test b/mysql-test/suite/multi_source/gtid_ignore_duplicates.test
index 3d2d151bd0d..cbc06920b41 100644
--- a/mysql-test/suite/multi_source/gtid_ignore_duplicates.test
+++ b/mysql-test/suite/multi_source/gtid_ignore_duplicates.test
@@ -173,6 +173,65 @@ SET default_master_connection = "a2b";
SELECT * FROM t1 WHERE a >= 10 ORDER BY a;
+--echo *** MDEV-33475: --gtid-ignore-duplicate can double-apply event in case of parallel replication retry
+
+# Create a bunch of transactions that will cause conflicts and retries.
+# The bug was that the retry code was not handling the --gtid-ignore-duplicates
+# option, so events could be doubly-applied.
+
+--connection server_2
+STOP SLAVE "c2b";
+SET default_master_connection = "c2b";
+--source include/wait_for_slave_to_stop.inc
+STOP SLAVE "a2b";
+SET default_master_connection = "a2b";
+--source include/wait_for_slave_to_stop.inc
+
+--connection server_1
+CREATE TABLE t2 (a INT PRIMARY KEY, b INT) ENGINE=InnoDB;
+BEGIN;
+--let $i= 0
+while ($i < 10) {
+ eval INSERT INTO t2 VALUES ($i, 0);
+ inc $i;
+}
+COMMIT;
+
+--let $i= 0
+while ($i < 10) {
+ BEGIN;
+ eval INSERT INTO t2 VALUES ($i+10, 100);
+ eval UPDATE t2 SET b=$i WHERE a<10;
+ eval INSERT INTO t2 VALUES ($i+20, 200);
+ COMMIT;
+ inc $i;
+}
+
+SELECT COUNT(*), SUM(a), SUM(b) FROM t2;
+--source include/save_master_gtid.inc
+
+--connection server_2
+SET @old_mode= @@GLOBAL.slave_parallel_mode;
+SET GLOBAL slave_parallel_mode=aggressive;
+SET default_master_connection = "a2b";
+START SLAVE;
+--source include/wait_for_slave_to_start.inc
+SET default_master_connection = "c2b";
+START SLAVE;
+--source include/wait_for_slave_to_start.inc
+
+--source include/sync_with_master_gtid.inc
+SELECT COUNT(*), SUM(a), SUM(b) FROM t2;
+
+--connection server_3
+--source include/sync_with_master_gtid.inc
+SELECT COUNT(*), SUM(a), SUM(b) FROM t2;
+
+--connection server_4
+--source include/sync_with_master_gtid.inc
+SELECT COUNT(*), SUM(a), SUM(b) FROM t2;
+
+
--echo *** Test also with not using parallel replication.
--connection server_1
@@ -414,6 +473,7 @@ SET GLOBAL gtid_domain_id=0;
--sorted_result
STOP ALL SLAVES;
SET GLOBAL slave_parallel_threads= @old_parallel;
+SET GLOBAL slave_parallel_mode= @old_mode;
SET GLOBAL gtid_ignore_duplicates= @old_ignore_duplicates;
--connection server_3
@@ -431,25 +491,25 @@ SET GLOBAL slave_parallel_threads= @old_parallel;
SET GLOBAL gtid_ignore_duplicates= @old_ignore_duplicates;
--connection server_1
-DROP TABLE t1;
+DROP TABLE t1, t2;
ALTER TABLE mysql.gtid_slave_pos ENGINE=Aria;
--source include/reset_master_slave.inc
--disconnect server_1
--connection server_2
-DROP TABLE t1;
+DROP TABLE t1, t2;
ALTER TABLE mysql.gtid_slave_pos ENGINE=Aria;
--source include/reset_master_slave.inc
--disconnect server_2
--connection server_3
-DROP TABLE t1;
+DROP TABLE t1, t2;
ALTER TABLE mysql.gtid_slave_pos ENGINE=Aria;
--source include/reset_master_slave.inc
--disconnect server_3
--connection server_4
-DROP TABLE t1;
+DROP TABLE t1, t2;
ALTER TABLE mysql.gtid_slave_pos ENGINE=Aria;
--source include/reset_master_slave.inc
--disconnect server_4
diff --git a/sql/rpl_parallel.cc b/sql/rpl_parallel.cc
index ac96d92eb5d..1c7f4dd1f93 100644
--- a/sql/rpl_parallel.cc
+++ b/sql/rpl_parallel.cc
@@ -868,7 +868,13 @@ retry_event_group(rpl_group_info *rgi, rpl_parallel_thread *rpt,
});
#endif
- rgi->cleanup_context(thd, 1);
+ /*
+ We are still applying the event group, even though we will roll it back
+ and retry it. So for --gtid-ignore-duplicates, keep ownership of the
+ domain during the retry so another master connection will not try to take
+ over and duplicate apply the same event group (MDEV-33475).
+ */
+ rgi->cleanup_context(thd, 1, 1 /* keep_domain_owner */);
wait_for_pending_deadlock_kill(thd, rgi);
thd->reset_killed();
thd->clear_error();
diff --git a/sql/rpl_rli.cc b/sql/rpl_rli.cc
index 95566b2f6c7..1af38be1787 100644
--- a/sql/rpl_rli.cc
+++ b/sql/rpl_rli.cc
@@ -2248,7 +2248,7 @@ delete_or_keep_event_post_apply(rpl_group_info *rgi,
}
-void rpl_group_info::cleanup_context(THD *thd, bool error)
+void rpl_group_info::cleanup_context(THD *thd, bool error, bool keep_domain_owner)
{
DBUG_ENTER("rpl_group_info::cleanup_context");
DBUG_PRINT("enter", ("error: %d", (int) error));
@@ -2298,7 +2298,7 @@ void rpl_group_info::cleanup_context(THD *thd, bool error)
Ensure we always release the domain for others to process, when using
--gtid-ignore-duplicates.
*/
- if (gtid_ignore_duplicate_state != GTID_DUPLICATE_NULL)
+ if (gtid_ignore_duplicate_state != GTID_DUPLICATE_NULL && !keep_domain_owner)
rpl_global_gtid_slave_state->release_domain_owner(this);
}
diff --git a/sql/rpl_rli.h b/sql/rpl_rli.h
index 9fc1a384355..91628bee3c7 100644
--- a/sql/rpl_rli.h
+++ b/sql/rpl_rli.h
@@ -917,7 +917,7 @@ struct rpl_group_info
}
void clear_tables_to_lock();
- void cleanup_context(THD *, bool);
+ void cleanup_context(THD *, bool, bool keep_domain_owner= false);
void slave_close_thread_tables(THD *);
void mark_start_commit_no_lock();
void mark_start_commit();
--
2.30.2
1
0

08 Mar '24
Remove incorrect deprecation.
Signed-off-by: Kristian Nielsen <knielsen(a)knielsen-hq.org>
---
.../suite/binlog_encryption/rpl_gtid_basic.result | 6 ------
.../suite/engines/funcs/r/rpl_change_master.result | 2 --
.../galera/r/galera_query_cache_invalidate.result | 2 --
mysql-test/suite/rpl/r/rpl_change_master.result | 5 -----
mysql-test/suite/rpl/r/rpl_gtid_basic.result | 6 ------
mysql-test/suite/rpl/r/rpl_gtid_errorhandling.result | 2 --
mysql-test/suite/rpl/r/rpl_gtid_mdev4820.result | 4 ----
.../suite/rpl/r/rpl_perfschema_connect_config.result | 2 --
mysql-test/suite/rpl/r/rpl_start_alter_options.result | 10 ----------
mysql-test/suite/rpl/r/rpl_using_gtid_default.result | 2 --
mysql-test/suite/rpl/t/rpl_change_master.test | 5 -----
sql/sql_repl.cc | 6 ------
12 files changed, 52 deletions(-)
diff --git a/mysql-test/suite/binlog_encryption/rpl_gtid_basic.result b/mysql-test/suite/binlog_encryption/rpl_gtid_basic.result
index 07ef909d8a6..0e066fc0418 100644
--- a/mysql-test/suite/binlog_encryption/rpl_gtid_basic.result
+++ b/mysql-test/suite/binlog_encryption/rpl_gtid_basic.result
@@ -69,8 +69,6 @@ INSERT INTO t2 VALUES (5, "i1a");
connection server_4;
CHANGE MASTER TO master_host = '127.0.0.1', master_port = MASTER_PORT,
MASTER_USE_GTID=CURRENT_POS;
-Warnings:
-Warning 1681 'master_use_gtid=current_pos' is deprecated and will be removed in a future release. Please use master_demote_to_slave=1 instead
include/start_slave.inc
SELECT * FROM t1 ORDER BY a;
a b
@@ -91,8 +89,6 @@ connection server_2;
include/stop_slave.inc
CHANGE MASTER TO master_host = '127.0.0.1', master_port = SERVER_MYPORT_4,
MASTER_USE_GTID=CURRENT_POS;
-Warnings:
-Warning 1681 'master_use_gtid=current_pos' is deprecated and will be removed in a future release. Please use master_demote_to_slave=1 instead
include/start_slave.inc
connection server_4;
UPDATE t2 SET b="j1a" WHERE a=5;
@@ -121,8 +117,6 @@ include/save_master_gtid.inc
connection server_3;
CHANGE MASTER TO master_host = '127.0.0.1', master_port = SERVER_MYPORT_4,
MASTER_USE_GTID=CURRENT_POS;
-Warnings:
-Warning 1681 'master_use_gtid=current_pos' is deprecated and will be removed in a future release. Please use master_demote_to_slave=1 instead
include/start_slave.inc
include/sync_with_master_gtid.inc
SELECT * FROM t2 ORDER BY a;
diff --git a/mysql-test/suite/engines/funcs/r/rpl_change_master.result b/mysql-test/suite/engines/funcs/r/rpl_change_master.result
index 88801b07bba..0713d8ef15b 100644
--- a/mysql-test/suite/engines/funcs/r/rpl_change_master.result
+++ b/mysql-test/suite/engines/funcs/r/rpl_change_master.result
@@ -28,7 +28,5 @@ MASTER_SSL_KEY='', MASTER_SSL_CRL='', MASTER_SSL_CRLPATH='';
CHANGE MASTER TO MASTER_USER='root', MASTER_PASSWORD='', MASTER_SSL=0;
"Usage of CURRENT_POS in CHANGE MASTER MASTER_USE_GTID is dreprecated.
CHANGE MASTER TO MASTER_USE_GTID=CURRENT_POS;
-Warnings:
-Warning 1681 'master_use_gtid=current_pos' is deprecated and will be removed in a future release. Please use master_demote_to_slave=1 instead
CHANGE MASTER TO MASTER_USE_GTID=SLAVE_POS;
include/rpl_end.inc
diff --git a/mysql-test/suite/galera/r/galera_query_cache_invalidate.result b/mysql-test/suite/galera/r/galera_query_cache_invalidate.result
index 4a6b61e4d80..fc23c0f1caf 100644
--- a/mysql-test/suite/galera/r/galera_query_cache_invalidate.result
+++ b/mysql-test/suite/galera/r/galera_query_cache_invalidate.result
@@ -8,8 +8,6 @@ connection node_4;
call mtr.add_suppression("WSREP: Ignoring server id for non bootstrap node.");
connection node_3;
CHANGE MASTER TO master_host='127.0.0.1', master_user='root', master_port=NODE_MYPORT_1, master_use_gtid=current_pos;;
-Warnings:
-Warning 1681 'master_use_gtid=current_pos' is deprecated and will be removed in a future release. Please use master_demote_to_slave=1 instead
START SLAVE;
include/wait_for_slave_to_start.inc
connection node_1;
diff --git a/mysql-test/suite/rpl/r/rpl_change_master.result b/mysql-test/suite/rpl/r/rpl_change_master.result
index 88801b07bba..48cec72d917 100644
--- a/mysql-test/suite/rpl/r/rpl_change_master.result
+++ b/mysql-test/suite/rpl/r/rpl_change_master.result
@@ -26,9 +26,4 @@ connection master;
CHANGE MASTER TO MASTER_USER='root', MASTER_SSL=0, MASTER_SSL_CA='', MASTER_SSL_CERT='',
MASTER_SSL_KEY='', MASTER_SSL_CRL='', MASTER_SSL_CRLPATH='';
CHANGE MASTER TO MASTER_USER='root', MASTER_PASSWORD='', MASTER_SSL=0;
-"Usage of CURRENT_POS in CHANGE MASTER MASTER_USE_GTID is dreprecated.
-CHANGE MASTER TO MASTER_USE_GTID=CURRENT_POS;
-Warnings:
-Warning 1681 'master_use_gtid=current_pos' is deprecated and will be removed in a future release. Please use master_demote_to_slave=1 instead
-CHANGE MASTER TO MASTER_USE_GTID=SLAVE_POS;
include/rpl_end.inc
diff --git a/mysql-test/suite/rpl/r/rpl_gtid_basic.result b/mysql-test/suite/rpl/r/rpl_gtid_basic.result
index fc7505e144f..32df09789cc 100644
--- a/mysql-test/suite/rpl/r/rpl_gtid_basic.result
+++ b/mysql-test/suite/rpl/r/rpl_gtid_basic.result
@@ -69,8 +69,6 @@ INSERT INTO t2 VALUES (5, "i1a");
connection server_4;
CHANGE MASTER TO master_host = '127.0.0.1', master_port = MASTER_PORT,
MASTER_USE_GTID=CURRENT_POS;
-Warnings:
-Warning 1681 'master_use_gtid=current_pos' is deprecated and will be removed in a future release. Please use master_demote_to_slave=1 instead
include/start_slave.inc
SELECT * FROM t1 ORDER BY a;
a b
@@ -91,8 +89,6 @@ connection server_2;
include/stop_slave.inc
CHANGE MASTER TO master_host = '127.0.0.1', master_port = SERVER_MYPORT_4,
MASTER_USE_GTID=CURRENT_POS;
-Warnings:
-Warning 1681 'master_use_gtid=current_pos' is deprecated and will be removed in a future release. Please use master_demote_to_slave=1 instead
include/start_slave.inc
connection server_4;
UPDATE t2 SET b="j1a" WHERE a=5;
@@ -121,8 +117,6 @@ include/save_master_gtid.inc
connection server_3;
CHANGE MASTER TO master_host = '127.0.0.1', master_port = SERVER_MYPORT_4,
MASTER_USE_GTID=CURRENT_POS;
-Warnings:
-Warning 1681 'master_use_gtid=current_pos' is deprecated and will be removed in a future release. Please use master_demote_to_slave=1 instead
include/start_slave.inc
include/sync_with_master_gtid.inc
SELECT * FROM t2 ORDER BY a;
diff --git a/mysql-test/suite/rpl/r/rpl_gtid_errorhandling.result b/mysql-test/suite/rpl/r/rpl_gtid_errorhandling.result
index a7cb710cc07..4c35d42d90a 100644
--- a/mysql-test/suite/rpl/r/rpl_gtid_errorhandling.result
+++ b/mysql-test/suite/rpl/r/rpl_gtid_errorhandling.result
@@ -75,8 +75,6 @@ INSERT INTO t1 VALUES (2);
SET sql_log_bin = 1;
INSERT INTO t1 VALUES (3);
CHANGE MASTER TO master_use_gtid=current_pos;
-Warnings:
-Warning 1681 'master_use_gtid=current_pos' is deprecated and will be removed in a future release. Please use master_demote_to_slave=1 instead
BEGIN;
SET GLOBAL gtid_slave_pos = "100-100-100";
ERROR 25000: You are not allowed to execute this command in a transaction
diff --git a/mysql-test/suite/rpl/r/rpl_gtid_mdev4820.result b/mysql-test/suite/rpl/r/rpl_gtid_mdev4820.result
index cea5aaaeacd..665fc536df6 100644
--- a/mysql-test/suite/rpl/r/rpl_gtid_mdev4820.result
+++ b/mysql-test/suite/rpl/r/rpl_gtid_mdev4820.result
@@ -45,8 +45,6 @@ SET GLOBAL gtid_slave_pos= '0-2-10';
connection server_1;
CHANGE MASTER TO master_host = '127.0.0.1', master_port = SERVER_MYPORT_2,
master_user= 'root', master_use_gtid=CURRENT_POS;
-Warnings:
-Warning 1681 'master_use_gtid=current_pos' is deprecated and will be removed in a future release. Please use master_demote_to_slave=1 instead
START SLAVE;
connection server_2;
INSERT INTO t1 VALUES (11);
@@ -76,8 +74,6 @@ connection server_2;
INSERT INTO t1 VALUES (22);
CHANGE MASTER TO master_host = '127.0.0.1', master_port = SERVER_MYPORT_1,
master_user= 'root', master_use_gtid=CURRENT_POS;
-Warnings:
-Warning 1681 'master_use_gtid=current_pos' is deprecated and will be removed in a future release. Please use master_demote_to_slave=1 instead
START SLAVE;
SET sql_log_bin= 0;
CALL mtr.add_suppression("which is not in the master's binlog. Since the master's binlog contains GTIDs with higher sequence numbers, it probably means that the slave has diverged");
diff --git a/mysql-test/suite/rpl/r/rpl_perfschema_connect_config.result b/mysql-test/suite/rpl/r/rpl_perfschema_connect_config.result
index 27cb29d3968..4ace84ffac4 100644
--- a/mysql-test/suite/rpl/r/rpl_perfschema_connect_config.result
+++ b/mysql-test/suite/rpl/r/rpl_perfschema_connect_config.result
@@ -87,8 +87,6 @@ include/assert.inc [Value returned by SSS and PS table for Using_Gtid should be
change master to
master_user = 'root',
master_use_gtid= CURRENT_POS;
-Warnings:
-Warning #### 'master_use_gtid=current_pos' is deprecated and will be removed in a future release. Please use master_demote_to_slave=1 instead
include/assert.inc [Value returned by SSS and PS table for Using_Gtid should be same.]
# 3) Test for Auto_position= SLAVE_POS
diff --git a/mysql-test/suite/rpl/r/rpl_start_alter_options.result b/mysql-test/suite/rpl/r/rpl_start_alter_options.result
index 4c6135aaf5e..30854b12be1 100644
--- a/mysql-test/suite/rpl/r/rpl_start_alter_options.result
+++ b/mysql-test/suite/rpl/r/rpl_start_alter_options.result
@@ -3,8 +3,6 @@ include/master-slave.inc
connection slave;
stop slave;
change master to master_use_gtid= current_pos;
-Warnings:
-Warning 1681 'master_use_gtid=current_pos' is deprecated and will be removed in a future release. Please use master_demote_to_slave=1 instead
SET GLOBAL slave_parallel_threads=4;
set global slave_parallel_mode=optimistic;
set global gtid_strict_mode=1;
@@ -96,8 +94,6 @@ include/start_slave.inc
connection slave;
stop slave;
change master to master_use_gtid= current_pos;
-Warnings:
-Warning 1681 'master_use_gtid=current_pos' is deprecated and will be removed in a future release. Please use master_demote_to_slave=1 instead
SET GLOBAL slave_parallel_threads=4;
set global slave_parallel_mode=optimistic;
set global gtid_strict_mode=1;
@@ -189,8 +185,6 @@ include/start_slave.inc
connection slave;
stop slave;
change master to master_use_gtid= current_pos;
-Warnings:
-Warning 1681 'master_use_gtid=current_pos' is deprecated and will be removed in a future release. Please use master_demote_to_slave=1 instead
SET GLOBAL slave_parallel_threads=4;
set global slave_parallel_mode=optimistic;
set global gtid_strict_mode=1;
@@ -321,8 +315,6 @@ include/start_slave.inc
connection slave;
stop slave;
change master to master_use_gtid= current_pos;
-Warnings:
-Warning 1681 'master_use_gtid=current_pos' is deprecated and will be removed in a future release. Please use master_demote_to_slave=1 instead
SET GLOBAL slave_parallel_threads=4;
set global slave_parallel_mode=optimistic;
set global gtid_strict_mode=1;
@@ -414,8 +406,6 @@ include/start_slave.inc
connection slave;
stop slave;
change master to master_use_gtid= current_pos;
-Warnings:
-Warning 1681 'master_use_gtid=current_pos' is deprecated and will be removed in a future release. Please use master_demote_to_slave=1 instead
SET GLOBAL slave_parallel_threads=4;
set global slave_parallel_mode=optimistic;
set global gtid_strict_mode=1;
diff --git a/mysql-test/suite/rpl/r/rpl_using_gtid_default.result b/mysql-test/suite/rpl/r/rpl_using_gtid_default.result
index e077bcd91a4..c0e9b3add00 100644
--- a/mysql-test/suite/rpl/r/rpl_using_gtid_default.result
+++ b/mysql-test/suite/rpl/r/rpl_using_gtid_default.result
@@ -58,8 +58,6 @@ include/start_slave.inc
# to its default of Slave_Pos after RESET SLAVE.
include/stop_slave.inc
CHANGE MASTER TO MASTER_USE_GTID=Current_Pos;
-Warnings:
-Warning 1681 'master_use_gtid=current_pos' is deprecated and will be removed in a future release. Please use master_demote_to_slave=1 instead
include/start_slave.inc
include/stop_slave.inc
RESET SLAVE;
diff --git a/mysql-test/suite/rpl/t/rpl_change_master.test b/mysql-test/suite/rpl/t/rpl_change_master.test
index 2758f9d6e27..992e23906e5 100644
--- a/mysql-test/suite/rpl/t/rpl_change_master.test
+++ b/mysql-test/suite/rpl/t/rpl_change_master.test
@@ -109,9 +109,4 @@ CHANGE MASTER TO MASTER_USER='root', MASTER_SSL=0, MASTER_SSL_CA='', MASTER_SSL_
CHANGE MASTER TO MASTER_USER='root', MASTER_PASSWORD='', MASTER_SSL=0;
-# MDEV-20122: Deprecate MASTER_USE_GTID=Current_Pos to favor new MASTER_DEMOTE_TO_SLAVE option
---echo "Usage of CURRENT_POS in CHANGE MASTER MASTER_USE_GTID is dreprecated.
-CHANGE MASTER TO MASTER_USE_GTID=CURRENT_POS;
-CHANGE MASTER TO MASTER_USE_GTID=SLAVE_POS;
-
--source include/rpl_end.inc
diff --git a/sql/sql_repl.cc b/sql/sql_repl.cc
index dc27ab9ff8b..c9fb3c6fd1d 100644
--- a/sql/sql_repl.cc
+++ b/sql/sql_repl.cc
@@ -3826,13 +3826,7 @@ bool change_master(THD* thd, Master_info* mi, bool *master_info_added)
if (lex_mi->use_gtid_opt == LEX_MASTER_INFO::LEX_GTID_SLAVE_POS)
mi->using_gtid= Master_info::USE_GTID_SLAVE_POS;
else if (lex_mi->use_gtid_opt == LEX_MASTER_INFO::LEX_GTID_CURRENT_POS)
- {
mi->using_gtid= Master_info::USE_GTID_CURRENT_POS;
- push_warning_printf(thd, Sql_condition::WARN_LEVEL_WARN,
- ER_WARN_DEPRECATED_SYNTAX_NO_REPLACEMENT,
- ER_THD(thd, ER_WARN_DEPRECATED_SYNTAX),
- "master_use_gtid=current_pos", "master_demote_to_slave=1");
- }
else if (lex_mi->use_gtid_opt == LEX_MASTER_INFO::LEX_GTID_NO ||
lex_mi->log_file_name || lex_mi->pos ||
lex_mi->relay_log_name || lex_mi->relay_log_pos)
--
2.30.2
1
0